KS
- The key type of the input keyed stream.IN1
- The input type of the keyed (non-broadcast) side.IN2
- The input type of the broadcast side.OUT
- The output type of the operator.@PublicEvolving public abstract class KeyedBroadcastProcessFunction<KS,IN1,IN2,OUT> extends BaseBroadcastProcessFunction
BroadcastConnectedStream
that
connects BroadcastStream
, i.e.
a stream with broadcast state, with a KeyedStream
.
The stream with the broadcast state can be created using the DataStream.broadcast(MapStateDescriptor[])
keyedStream.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 an argument (among others) a context that
allows it to read/write to the broadcast state and also apply a transformation to all (local)
keyed states, while the processElement()
has read-only access to the broadcast state, but
can read/write to the keyed state and register timers.
Modifier and Type | Class and Description |
---|---|
class |
KeyedBroadcastProcessFunction.Context
A
context available to the broadcast side of a
BroadcastConnectedStream . |
class |
KeyedBroadcastProcessFunction.OnTimerContext
Information available in an invocation of
onTimer(long, OnTimerContext, Collector) . |
class |
KeyedBroadcastProcessFunction.ReadOnlyContext
A
context available to the keyed stream side of
a BroadcastConnectedStream (if any). |
Constructor and Description |
---|
KeyedBroadcastProcessFunction() |
Modifier and Type | Method and Description |
---|---|
void |
onTimer(long timestamp,
KeyedBroadcastProcessFunction.OnTimerContext ctx,
Collector<OUT> out)
Called when a timer set using
TimerService fires. |
abstract void |
processBroadcastElement(IN2 value,
KeyedBroadcastProcessFunction.Context ctx,
Collector<OUT> out)
This method is called for each element in the
broadcast stream . |
abstract void |
processElement(IN1 value,
KeyedBroadcastProcessFunction.ReadOnlyContext ctx,
Collector<OUT> out)
This method is called for each element in the (non-broadcast)
keyed stream . |
close, getIterationRuntimeContext, getRuntimeContext, open, setRuntimeContext
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
open
public abstract void processElement(IN1 value, KeyedBroadcastProcessFunction.ReadOnlyContext ctx, Collector<OUT> out) throws Exception
keyed stream
.
It 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. In addition,
it can get a TimerService
for registering timers and querying the time. 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 KeyedBroadcastProcessFunction.ReadOnlyContext
that allows querying the timestamp of the element,
querying the current processing/event time and iterating the broadcast state with
read-only access. 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, KeyedBroadcastProcessFunction.Context ctx, Collector<OUT> out) throws Exception
broadcast stream
.
It 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
. In addition, it can
register a function
to be applied to all keyed states on the local
partition. These can be done through the provided KeyedBroadcastProcessFunction.Context
. The context is only valid
during the invocation of this method, do not store it.
value
- The stream element.ctx
- A KeyedBroadcastProcessFunction.Context
that allows querying the timestamp of the element, querying the
current processing/event time and updating the broadcast state. In addition, it allows
the registration of a function
to be applied to all keyed
state with a given StateDescriptor
on the local partition. 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 void onTimer(long timestamp, KeyedBroadcastProcessFunction.OnTimerContext ctx, Collector<OUT> out) throws Exception
TimerService
fires.timestamp
- The timestamp of the firing timer.ctx
- An KeyedBroadcastProcessFunction.OnTimerContext
that allows querying the timestamp of the firing timer,
querying the current processing/event time, iterating the broadcast state with
read-only access, querying the TimeDomain
of the firing timer and getting
a TimerService
for registering timers and querying the time. The context is only
valid during the invocation of this method, do not store it.out
- The collector for returning result values.Exception
- This method may throw exceptions. Throwing an exception will cause the
operation to fail and may trigger recovery.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.