IN1
- The input type of the non-broadcast side.IN2
- The input type of the broadcast side.OUT
- The output type of the operator.@PublicEvolving public abstract class BroadcastProcessFunction<IN1,IN2,OUT> extends BaseBroadcastProcessFunction
BroadcastConnectedStream
that
connects BroadcastStream
, i.e.
a stream with broadcast state, with a non-keyed DataStream
.
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:
processBroadcastElement(Object, Context, Collector)
which will be applied to
each element in the broadcast side
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 the processElement()
has read-only
access to the broadcast state.
Modifier and Type | Class and Description |
---|---|
class |
BroadcastProcessFunction.Context
A
context available to the broadcast side of a
BroadcastConnectedStream . |
class |
BroadcastProcessFunction.ReadOnlyContext
A
context available to the non-keyed side of a
BroadcastConnectedStream (if any). |
Constructor and Description |
---|
BroadcastProcessFunction() |
Modifier and Type | Method and Description |
---|---|
abstract void |
processBroadcastElement(IN2 value,
BroadcastProcessFunction.Context ctx,
Collector<OUT> out)
This method is called for each element in the
broadcast 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 . |
close, getIterationRuntimeContext, getRuntimeContext, open, setRuntimeContext
public abstract void processElement(IN1 value, BroadcastProcessFunction.ReadOnlyContext ctx, Collector<OUT> out) throws Exception
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.
value
- The stream element.ctx
- A BroadcastProcessFunction.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 toException
- The function may throw exceptions which cause the streaming program to fail
and go into recovery.public abstract void processBroadcastElement(IN2 value, BroadcastProcessFunction.Context ctx, Collector<OUT> out) throws Exception
broadcast 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 internal broadcast state
. These can be done through
the provided BroadcastProcessFunction.Context
. The context is only valid during the invocation of this method,
do not store it.
value
- The stream element.ctx
- A BroadcastProcessFunction.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 toException
- The function may throw exceptions which cause the streaming program to fail
and go into recovery.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.