Class BroadcastProcessFunction<IN1,IN2,OUT>
- java.lang.Object
-
- org.apache.flink.api.common.functions.AbstractRichFunction
-
- org.apache.flink.streaming.api.functions.co.BaseBroadcastProcessFunction
-
- org.apache.flink.streaming.api.functions.co.BroadcastProcessFunction<IN1,IN2,OUT>
-
- Type Parameters:
IN1
- The input type of the non-broadcast side.IN2
- The input type of the broadcast side.OUT
- The output type of the operator.
- All Implemented Interfaces:
Serializable
,Function
,RichFunction
@PublicEvolving public abstract class BroadcastProcessFunction<IN1,IN2,OUT> extends BaseBroadcastProcessFunction
A function to be applied to aBroadcastConnectedStream
that connectsBroadcastStream
, i.e. a stream with broadcast state, with a non-keyedDataStream
.The stream with the broadcast state can be created using the
DataStream.broadcast(MapStateDescriptor[])
stream.broadcast(MapStateDescriptor)} method.The user has to implement two methods:
- the
#processBroadcastElement(Object, Context, Collector)
which will be applied to each element in the broadcast side - and the
#processElement(Object, ReadOnlyContext, Collector)
which will be applied to the non-broadcasted/keyed side.
The
processElementOnBroadcastSide()
takes as argument (among others) a context that allows it to read/write to the broadcast state, while theprocessElement()
has read-only access to the broadcast state.- See Also:
- Serialized Form
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description class
BroadcastProcessFunction.Context
Acontext
available to the broadcast side of aBroadcastConnectedStream
.class
BroadcastProcessFunction.ReadOnlyContext
Acontext
available to the non-keyed side of aBroadcastConnectedStream
(if any).
-
Constructor Summary
Constructors Constructor Description BroadcastProcessFunction()
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method Description abstract void
processBroadcastElement(IN2 value, BroadcastProcessFunction.Context ctx, Collector<OUT> out)
This method is called for each element in thebroadcast stream
.abstract void
processElement(IN1 value, BroadcastProcessFunction.ReadOnlyContext ctx, Collector<OUT> out)
This method is called for each element in the (non-broadcast)data stream
.-
Methods inherited from class org.apache.flink.api.common.functions.AbstractRichFunction
close, getIterationRuntimeContext, getRuntimeContext, open, setRuntimeContext
-
-
-
-
Method Detail
-
processElement
public abstract void processElement(IN1 value, BroadcastProcessFunction.ReadOnlyContext ctx, Collector<OUT> out) throws Exception
This method is called for each element in the (non-broadcast)data stream
.This function can output zero or more elements using the
Collector
parameter, query the current processing/event time, and also query and update the local keyed state. Finally, it has read-only access to the broadcast state. The context is only valid during the invocation of this method, do not store it.- Parameters:
value
- The stream element.ctx
- ABroadcastProcessFunction.ReadOnlyContext
that allows querying the timestamp of the element, querying the current processing/event time and updating the broadcast state. The context is only valid during the invocation of this method, do not store it.out
- The collector to emit resulting elements to- Throws:
Exception
- The function may throw exceptions which cause the streaming program to fail and go into recovery.
-
processBroadcastElement
public abstract void processBroadcastElement(IN2 value, BroadcastProcessFunction.Context ctx, Collector<OUT> out) throws Exception
This method is called for each element in thebroadcast stream
.This function can output zero or more elements using the
Collector
parameter, query the current processing/event time, and also query and update the internalbroadcast state
. These can be done through the providedBroadcastProcessFunction.Context
. The context is only valid during the invocation of this method, do not store it.- Parameters:
value
- The stream element.ctx
- ABroadcastProcessFunction.Context
that allows querying the timestamp of the element, querying the current processing/event time and updating the broadcast state. The context is only valid during the invocation of this method, do not store it.out
- The collector to emit resulting elements to- Throws:
Exception
- The function may throw exceptions which cause the streaming program to fail and go into recovery.
-
-