Class KeyedCoProcessFunction<K,IN1,IN2,OUT>
- java.lang.Object
-
- org.apache.flink.api.common.functions.AbstractRichFunction
-
- org.apache.flink.streaming.api.functions.co.KeyedCoProcessFunction<K,IN1,IN2,OUT>
-
- Type Parameters:
K
- Type of the key.IN1
- Type of the first input.IN2
- Type of the second input.OUT
- Output type.
- All Implemented Interfaces:
Serializable
,Function
,RichFunction
- Direct Known Subclasses:
ProcTimeIntervalJoin
,RowTimeIntervalJoin
@PublicEvolving public abstract class KeyedCoProcessFunction<K,IN1,IN2,OUT> extends AbstractRichFunction
A function that processes elements of two keyed streams and produces a single output one.The function will be called for every element in the input streams and can produce zero or more output elements. Contrary to the
CoFlatMapFunction
, this function can also query the time (both event and processing) and set timers, through the providedKeyedCoProcessFunction.Context
. When reacting to the firing of set timers the function can emit yet more elements.An example use-case for connected streams would be the application of a set of rules that change over time (
stream A
) to the elements contained in another stream (streamB
). The rules contained instream A
can be stored in the state and wait for new elements to arrive onstream B
. Upon reception of a new element onstream B
, the function can now apply the previously stored rules to the element and directly emit a result, and/or register a timer that will trigger an action in the future.- See Also:
- Serialized Form
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description class
KeyedCoProcessFunction.Context
Information available in an invocation of#processElement1(Object, Context, Collector)
/#processElement2(Object, Context, Collector)
or#onTimer(long, OnTimerContext, Collector)
.class
KeyedCoProcessFunction.OnTimerContext
Information available in an invocation of#onTimer(long, OnTimerContext, Collector)
.
-
Constructor Summary
Constructors Constructor Description KeyedCoProcessFunction()
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
onTimer(long timestamp, KeyedCoProcessFunction.OnTimerContext ctx, Collector<OUT> out)
Called when a timer set usingTimerService
fires.abstract void
processElement1(IN1 value, KeyedCoProcessFunction.Context ctx, Collector<OUT> out)
This method is called for each element in the first of the connected streams.abstract void
processElement2(IN2 value, KeyedCoProcessFunction.Context ctx, Collector<OUT> out)
This method is called for each element in the second of the connected streams.-
Methods inherited from class org.apache.flink.api.common.functions.AbstractRichFunction
close, getIterationRuntimeContext, getRuntimeContext, open, setRuntimeContext
-
-
-
-
Method Detail
-
processElement1
public abstract void processElement1(IN1 value, KeyedCoProcessFunction.Context ctx, Collector<OUT> out) throws Exception
This method is called for each element in the first of the connected streams.This function can output zero or more elements using the
Collector
parameter and also update internal state or set timers using theKeyedCoProcessFunction.Context
parameter.- Parameters:
value
- The stream elementctx
- AKeyedCoProcessFunction.Context
that allows querying the timestamp of the element, querying theTimeDomain
of the firing timer and getting aTimerService
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 to emit resulting elements to- Throws:
Exception
- The function may throw exceptions which cause the streaming program to fail and go into recovery.
-
processElement2
public abstract void processElement2(IN2 value, KeyedCoProcessFunction.Context ctx, Collector<OUT> out) throws Exception
This method is called for each element in the second of the connected streams.This function can output zero or more elements using the
Collector
parameter and also update internal state or set timers using theKeyedCoProcessFunction.Context
parameter.- Parameters:
value
- The stream elementctx
- AKeyedCoProcessFunction.Context
that allows querying the timestamp of the element, querying theTimeDomain
of the firing timer and getting aTimerService
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 to emit resulting elements to- Throws:
Exception
- The function may throw exceptions which cause the streaming program to fail and go into recovery.
-
onTimer
public void onTimer(long timestamp, KeyedCoProcessFunction.OnTimerContext ctx, Collector<OUT> out) throws Exception
Called when a timer set usingTimerService
fires.- Parameters:
timestamp
- The timestamp of the firing timer.ctx
- AnKeyedCoProcessFunction.OnTimerContext
that allows querying the timestamp of the firing timer, querying theTimeDomain
of the firing timer and getting aTimerService
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.- Throws:
Exception
- This method may throw exceptions. Throwing an exception will cause the operation to fail and may trigger recovery.
-
-