Class BroadcastConnectedStream<IN1,IN2>
- java.lang.Object
-
- org.apache.flink.streaming.api.datastream.BroadcastConnectedStream<IN1,IN2>
-
- Type Parameters:
IN1
- The input type of the non-broadcast side.IN2
- The input type of the broadcast side.
@PublicEvolving public class BroadcastConnectedStream<IN1,IN2> extends Object
A BroadcastConnectedStream represents the result of connecting a keyed or non-keyed stream, with aBroadcastStream
withbroadcast state(s)
. As in the case ofConnectedStreams
these streams are useful for cases where operations on one stream directly affect the operations on the other stream, usually via shared state between the streams.An example for the use of such connected streams would be to apply rules that change over time onto another, possibly keyed stream. The stream with the broadcast state has the rules, and will store them in the broadcast state, while the other stream will contain the elements to apply the rules to. By broadcasting the rules, these will be available in all parallel instances, and can be applied to all partitions of the other stream.
-
-
Constructor Summary
Constructors Modifier Constructor Description protected
BroadcastConnectedStream(StreamExecutionEnvironment env, DataStream<IN1> input1, BroadcastStream<IN2> input2, List<MapStateDescriptor<?,?>> broadcastStateDescriptors)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description protected <F> F
clean(F f)
StreamExecutionEnvironment
getExecutionEnvironment()
DataStream<IN1>
getFirstInput()
Returns the non-broadcastDataStream
.BroadcastStream<IN2>
getSecondInput()
Returns theBroadcastStream
.TypeInformation<IN1>
getType1()
Gets the type of the first input.TypeInformation<IN2>
getType2()
Gets the type of the second input.<OUT> SingleOutputStreamOperator<OUT>
process(BroadcastProcessFunction<IN1,IN2,OUT> function)
Assumes as inputs aBroadcastStream
and a non-keyedDataStream
and applies the givenBroadcastProcessFunction
on them, thereby creating a transformed output stream.<OUT> SingleOutputStreamOperator<OUT>
process(BroadcastProcessFunction<IN1,IN2,OUT> function, TypeInformation<OUT> outTypeInfo)
Assumes as inputs aBroadcastStream
and a non-keyedDataStream
and applies the givenBroadcastProcessFunction
on them, thereby creating a transformed output stream.<KEY,OUT>
SingleOutputStreamOperator<OUT>process(KeyedBroadcastProcessFunction<KEY,IN1,IN2,OUT> function)
Assumes as inputs aBroadcastStream
and aKeyedStream
and applies the givenKeyedBroadcastProcessFunction
on them, thereby creating a transformed output stream.<KEY,OUT>
SingleOutputStreamOperator<OUT>process(KeyedBroadcastProcessFunction<KEY,IN1,IN2,OUT> function, TypeInformation<OUT> outTypeInfo)
Assumes as inputs aBroadcastStream
and aKeyedStream
and applies the givenKeyedBroadcastProcessFunction
on them, thereby creating a transformed output stream.
-
-
-
Constructor Detail
-
BroadcastConnectedStream
protected BroadcastConnectedStream(StreamExecutionEnvironment env, DataStream<IN1> input1, BroadcastStream<IN2> input2, List<MapStateDescriptor<?,?>> broadcastStateDescriptors)
-
-
Method Detail
-
getExecutionEnvironment
public StreamExecutionEnvironment getExecutionEnvironment()
-
getFirstInput
public DataStream<IN1> getFirstInput()
Returns the non-broadcastDataStream
.- Returns:
- The stream which, by convention, is not broadcasted.
-
getSecondInput
public BroadcastStream<IN2> getSecondInput()
Returns theBroadcastStream
.- Returns:
- The stream which, by convention, is the broadcast one.
-
getType1
public TypeInformation<IN1> getType1()
Gets the type of the first input.- Returns:
- The type of the first input
-
getType2
public TypeInformation<IN2> getType2()
Gets the type of the second input.- Returns:
- The type of the second input
-
process
@PublicEvolving public <KEY,OUT> SingleOutputStreamOperator<OUT> process(KeyedBroadcastProcessFunction<KEY,IN1,IN2,OUT> function)
Assumes as inputs aBroadcastStream
and aKeyedStream
and applies the givenKeyedBroadcastProcessFunction
on them, thereby creating a transformed output stream.- Type Parameters:
KEY
- The type of the keys in the keyed stream.OUT
- The type of the output elements.- Parameters:
function
- TheKeyedBroadcastProcessFunction
that is called for each element in the stream.- Returns:
- The transformed
DataStream
.
-
process
@PublicEvolving public <KEY,OUT> SingleOutputStreamOperator<OUT> process(KeyedBroadcastProcessFunction<KEY,IN1,IN2,OUT> function, TypeInformation<OUT> outTypeInfo)
Assumes as inputs aBroadcastStream
and aKeyedStream
and applies the givenKeyedBroadcastProcessFunction
on them, thereby creating a transformed output stream.- Type Parameters:
KEY
- The type of the keys in the keyed stream.OUT
- The type of the output elements.- Parameters:
function
- TheKeyedBroadcastProcessFunction
that is called for each element in the stream.outTypeInfo
- The type of the output elements.- Returns:
- The transformed
DataStream
.
-
process
@PublicEvolving public <OUT> SingleOutputStreamOperator<OUT> process(BroadcastProcessFunction<IN1,IN2,OUT> function)
Assumes as inputs aBroadcastStream
and a non-keyedDataStream
and applies the givenBroadcastProcessFunction
on them, thereby creating a transformed output stream.- Type Parameters:
OUT
- The type of the output elements.- Parameters:
function
- TheBroadcastProcessFunction
that is called for each element in the stream.- Returns:
- The transformed
DataStream
.
-
process
@PublicEvolving public <OUT> SingleOutputStreamOperator<OUT> process(BroadcastProcessFunction<IN1,IN2,OUT> function, TypeInformation<OUT> outTypeInfo)
Assumes as inputs aBroadcastStream
and a non-keyedDataStream
and applies the givenBroadcastProcessFunction
on them, thereby creating a transformed output stream.- Type Parameters:
OUT
- The type of the output elements.- Parameters:
function
- TheBroadcastProcessFunction
that is called for each element in the stream.outTypeInfo
- The type of the output elements.- Returns:
- The transformed
DataStream
.
-
clean
protected <F> F clean(F f)
-
-