IN1
- Type of the first input data steam.IN2
- Type of the second input data stream.@Public public class ConnectedStreams<IN1,IN2> extends Object
An example for the use of connected streams would be to apply rules that change over time onto another stream. One of the connected streams has the rules, the other stream the elements to apply the rules to. The operation on the connected stream maintains the current set of rules in the state. It may receive either a rule update and update the state or a data element and apply the rules in the state to the element.
The connected stream can be conceptually viewed as a union stream of an Either type, that holds either the first stream's type or the second stream's type.
Modifier and Type | Field and Description |
---|---|
protected StreamExecutionEnvironment |
environment |
protected DataStream<IN1> |
inputStream1 |
protected DataStream<IN2> |
inputStream2 |
Modifier | Constructor and Description |
---|---|
protected |
ConnectedStreams(StreamExecutionEnvironment env,
DataStream<IN1> input1,
DataStream<IN2> input2) |
Modifier and Type | Method and Description |
---|---|
<R> SingleOutputStreamOperator<R> |
flatMap(CoFlatMapFunction<IN1,IN2,R> coFlatMapper)
Applies a CoFlatMap transformation on a
ConnectedStreams and maps the output to a
common type. |
<R> SingleOutputStreamOperator<R> |
flatMap(CoFlatMapFunction<IN1,IN2,R> coFlatMapper,
TypeInformation<R> outputType)
Applies a CoFlatMap transformation on a
ConnectedStreams and maps the output to a
common type. |
StreamExecutionEnvironment |
getExecutionEnvironment() |
DataStream<IN1> |
getFirstInput()
Returns the first
DataStream . |
DataStream<IN2> |
getSecondInput()
Returns the second
DataStream . |
TypeInformation<IN1> |
getType1()
Gets the type of the first input.
|
TypeInformation<IN2> |
getType2()
Gets the type of the second input.
|
ConnectedStreams<IN1,IN2> |
keyBy(int[] keyPositions1,
int[] keyPositions2)
KeyBy operation for connected data stream.
|
ConnectedStreams<IN1,IN2> |
keyBy(int keyPosition1,
int keyPosition2)
KeyBy operation for connected data stream.
|
<KEY> ConnectedStreams<IN1,IN2> |
keyBy(KeySelector<IN1,KEY> keySelector1,
KeySelector<IN2,KEY> keySelector2)
KeyBy operation for connected data stream.
|
<KEY> ConnectedStreams<IN1,IN2> |
keyBy(KeySelector<IN1,KEY> keySelector1,
KeySelector<IN2,KEY> keySelector2,
TypeInformation<KEY> keyType)
KeyBy operation for connected data stream.
|
ConnectedStreams<IN1,IN2> |
keyBy(String[] fields1,
String[] fields2)
KeyBy operation for connected data stream using key expressions. the elements of input1 and
input2 according to fields1 and fields2.
|
ConnectedStreams<IN1,IN2> |
keyBy(String field1,
String field2)
KeyBy operation for connected data stream using key expressions.
|
<R> SingleOutputStreamOperator<R> |
map(CoMapFunction<IN1,IN2,R> coMapper)
Applies a CoMap transformation on a
ConnectedStreams and maps the output to a common
type. |
<R> SingleOutputStreamOperator<R> |
map(CoMapFunction<IN1,IN2,R> coMapper,
TypeInformation<R> outputType)
Applies a CoMap transformation on a
ConnectedStreams and maps the output to a common
type. |
<R> SingleOutputStreamOperator<R> |
process(CoProcessFunction<IN1,IN2,R> coProcessFunction)
Applies the given
CoProcessFunction on the connected input streams, thereby creating
a transformed output stream. |
<R> SingleOutputStreamOperator<R> |
process(CoProcessFunction<IN1,IN2,R> coProcessFunction,
TypeInformation<R> outputType)
Applies the given
CoProcessFunction on the connected input streams, thereby creating
a transformed output stream. |
<K,R> SingleOutputStreamOperator<R> |
process(KeyedCoProcessFunction<K,IN1,IN2,R> keyedCoProcessFunction)
Applies the given
KeyedCoProcessFunction on the connected input keyed streams,
thereby creating a transformed output stream. |
<K,R> SingleOutputStreamOperator<R> |
process(KeyedCoProcessFunction<K,IN1,IN2,R> keyedCoProcessFunction,
TypeInformation<R> outputType)
Applies the given
KeyedCoProcessFunction on the connected input streams, thereby
creating a transformed output stream. |
<R> SingleOutputStreamOperator<R> |
transform(String functionName,
TypeInformation<R> outTypeInfo,
TwoInputStreamOperator<IN1,IN2,R> operator) |
<R> SingleOutputStreamOperator<R> |
transform(String functionName,
TypeInformation<R> outTypeInfo,
TwoInputStreamOperatorFactory<IN1,IN2,R> operatorFactory) |
protected final StreamExecutionEnvironment environment
protected final DataStream<IN1> inputStream1
protected final DataStream<IN2> inputStream2
protected ConnectedStreams(StreamExecutionEnvironment env, DataStream<IN1> input1, DataStream<IN2> input2)
public StreamExecutionEnvironment getExecutionEnvironment()
public DataStream<IN1> getFirstInput()
DataStream
.public DataStream<IN2> getSecondInput()
DataStream
.public TypeInformation<IN1> getType1()
public TypeInformation<IN2> getType2()
public ConnectedStreams<IN1,IN2> keyBy(int keyPosition1, int keyPosition2)
keyPosition1
- The field used to compute the hashcode of the elements in the first input
stream.keyPosition2
- The field used to compute the hashcode of the elements in the second
input stream.ConnectedStreams
public ConnectedStreams<IN1,IN2> keyBy(int[] keyPositions1, int[] keyPositions2)
keyPositions1
- The fields used to group the first input stream.keyPositions2
- The fields used to group the second input stream.ConnectedStreams
public ConnectedStreams<IN1,IN2> keyBy(String field1, String field2)
DataStream
S underlying
type. A dot can be used to drill down into objects, as in "field1.getInnerField2()"
.field1
- The grouping expression for the first inputfield2
- The grouping expression for the second inputConnectedStreams
public ConnectedStreams<IN1,IN2> keyBy(String[] fields1, String[] fields2)
DataStream
S underlying type. A dot
can be used to drill down into objects, as in "field1.getInnerField2()"
.fields1
- The grouping expressions for the first inputfields2
- The grouping expressions for the second inputConnectedStreams
public <KEY> ConnectedStreams<IN1,IN2> keyBy(KeySelector<IN1,KEY> keySelector1, KeySelector<IN2,KEY> keySelector2)
keySelector1
- The KeySelector
used for grouping the first inputkeySelector2
- The KeySelector
used for grouping the second inputConnectedStreams
public <KEY> ConnectedStreams<IN1,IN2> keyBy(KeySelector<IN1,KEY> keySelector1, KeySelector<IN2,KEY> keySelector2, TypeInformation<KEY> keyType)
keySelector1
- The KeySelector
used for grouping the first inputkeySelector2
- The KeySelector
used for grouping the second inputkeyType
- The type information of the common key type.ConnectedStreams
public <R> SingleOutputStreamOperator<R> map(CoMapFunction<IN1,IN2,R> coMapper)
ConnectedStreams
and maps the output to a common
type. The transformation calls a CoMapFunction.map1(IN1)
for each element of the first
input and CoMapFunction.map2(IN2)
for each element of the second input. Each CoMapFunction
call returns exactly one element.coMapper
- The CoMapFunction used to jointly transform the two input DataStreamsDataStream
public <R> SingleOutputStreamOperator<R> map(CoMapFunction<IN1,IN2,R> coMapper, TypeInformation<R> outputType)
ConnectedStreams
and maps the output to a common
type. The transformation calls a CoMapFunction.map1(IN1)
for each element of the first
input and CoMapFunction.map2(IN2)
for each element of the second input. Each CoMapFunction
call returns exactly one element.coMapper
- The CoMapFunction used to jointly transform the two input DataStreamsoutputType
- TypeInformation
for the result type of the function.DataStream
public <R> SingleOutputStreamOperator<R> flatMap(CoFlatMapFunction<IN1,IN2,R> coFlatMapper)
ConnectedStreams
and maps the output to a
common type. The transformation calls a CoFlatMapFunction.flatMap1(IN1, org.apache.flink.util.Collector<OUT>)
for each element
of the first input and CoFlatMapFunction.flatMap2(IN2, org.apache.flink.util.Collector<OUT>)
for each element of the second
input. Each CoFlatMapFunction call returns any number of elements including none.coFlatMapper
- The CoFlatMapFunction used to jointly transform the two input DataStreamsDataStream
public <R> SingleOutputStreamOperator<R> flatMap(CoFlatMapFunction<IN1,IN2,R> coFlatMapper, TypeInformation<R> outputType)
ConnectedStreams
and maps the output to a
common type. The transformation calls a CoFlatMapFunction.flatMap1(IN1, org.apache.flink.util.Collector<OUT>)
for each element
of the first input and CoFlatMapFunction.flatMap2(IN2, org.apache.flink.util.Collector<OUT>)
for each element of the second
input. Each CoFlatMapFunction call returns any number of elements including none.coFlatMapper
- The CoFlatMapFunction used to jointly transform the two input DataStreamsoutputType
- TypeInformation
for the result type of the function.DataStream
@PublicEvolving public <R> SingleOutputStreamOperator<R> process(CoProcessFunction<IN1,IN2,R> coProcessFunction)
CoProcessFunction
on the connected input streams, thereby creating
a transformed output stream.
The function will be called for every element in the input streams and can produce zero or
more output elements. Contrary to the flatMap(CoFlatMapFunction)
function, this
function can also query the time and set timers. When reacting to the firing of set timers
the function can directly emit elements and/or register yet more timers.
R
- The type of elements emitted by the CoProcessFunction
.coProcessFunction
- The CoProcessFunction
that is called for each element in the
stream.DataStream
.@Internal public <R> SingleOutputStreamOperator<R> process(CoProcessFunction<IN1,IN2,R> coProcessFunction, TypeInformation<R> outputType)
CoProcessFunction
on the connected input streams, thereby creating
a transformed output stream.
The function will be called for every element in the input streams and can produce zero or
more output elements. Contrary to the flatMap(CoFlatMapFunction)
function, this
function can also query the time and set timers. When reacting to the firing of set timers
the function can directly emit elements and/or register yet more timers.
R
- The type of elements emitted by the CoProcessFunction
.coProcessFunction
- The CoProcessFunction
that is called for each element in the
stream.DataStream
.@PublicEvolving public <K,R> SingleOutputStreamOperator<R> process(KeyedCoProcessFunction<K,IN1,IN2,R> keyedCoProcessFunction)
KeyedCoProcessFunction
on the connected input keyed streams,
thereby creating a transformed output stream.
The function will be called for every element in the input keyed streams and can produce
zero or more output elements. Contrary to the flatMap(CoFlatMapFunction)
function,
this function can also query the time and set timers. When reacting to the firing of set
timers the function can directly emit elements and/or register yet more timers.
R
- The type of elements emitted by the CoProcessFunction
.keyedCoProcessFunction
- The KeyedCoProcessFunction
that is called for each
element in the stream.DataStream
.@Internal public <K,R> SingleOutputStreamOperator<R> process(KeyedCoProcessFunction<K,IN1,IN2,R> keyedCoProcessFunction, TypeInformation<R> outputType)
KeyedCoProcessFunction
on the connected input streams, thereby
creating a transformed output stream.
The function will be called for every element in the input streams and can produce zero or
more output elements. Contrary to the flatMap(CoFlatMapFunction)
function, this
function can also query the time and set timers. When reacting to the firing of set timers
the function can directly emit elements and/or register yet more timers.
R
- The type of elements emitted by the CoProcessFunction
.keyedCoProcessFunction
- The KeyedCoProcessFunction
that is called for each
element in the stream.DataStream
.@PublicEvolving public <R> SingleOutputStreamOperator<R> transform(String functionName, TypeInformation<R> outTypeInfo, TwoInputStreamOperator<IN1,IN2,R> operator)
@PublicEvolving public <R> SingleOutputStreamOperator<R> transform(String functionName, TypeInformation<R> outTypeInfo, TwoInputStreamOperatorFactory<IN1,IN2,R> operatorFactory)
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.