Class ProcessFunction<I,O>
- java.lang.Object
-
- org.apache.flink.api.common.functions.AbstractRichFunction
-
- org.apache.flink.streaming.api.functions.ProcessFunction<I,O>
-
- Type Parameters:
I
- Type of the input elements.O
- Type of the output elements.
- All Implemented Interfaces:
Serializable
,Function
,RichFunction
- Direct Known Subclasses:
LookupJoinRunner
,PythonConnectorUtils.RowRowMapper
,SideOutputExample.Tokenizer
@PublicEvolving public abstract class ProcessFunction<I,O> extends AbstractRichFunction
A function that processes elements of a stream.For every element in the input stream
#processElement(Object, Context, Collector)
is invoked. This can produce zero or more elements as output. Implementations can also query the time and set timers through the providedProcessFunction.Context
. For firing timers#onTimer(long, OnTimerContext, Collector)
will be invoked. This can again produce zero or more elements as output and register further timers.NOTE: Access to keyed state and timers (which are also scoped to a key) is only available if the
ProcessFunction
is applied on aKeyedStream
.NOTE: A
ProcessFunction
is always aRichFunction
. Therefore, access to theRuntimeContext
is always available and setup and teardown methods can be implemented. SeeRichFunction.open(OpenContext)
andRichFunction.close()
.- See Also:
- Serialized Form
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description class
ProcessFunction.Context
Information available in an invocation of#processElement(Object, Context, Collector)
or#onTimer(long, OnTimerContext, Collector)
.class
ProcessFunction.OnTimerContext
Information available in an invocation of#onTimer(long, OnTimerContext, Collector)
.
-
Constructor Summary
Constructors Constructor Description ProcessFunction()
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
onTimer(long timestamp, ProcessFunction.OnTimerContext ctx, Collector<O> out)
Called when a timer set usingTimerService
fires.abstract void
processElement(I value, ProcessFunction.Context ctx, Collector<O> out)
Process one element from the input stream.-
Methods inherited from class org.apache.flink.api.common.functions.AbstractRichFunction
close, getIterationRuntimeContext, getRuntimeContext, open, setRuntimeContext
-
-
-
-
Method Detail
-
processElement
public abstract void processElement(I value, ProcessFunction.Context ctx, Collector<O> out) throws Exception
Process one element from the input stream.This function can output zero or more elements using the
Collector
parameter and also update internal state or set timers using theProcessFunction.Context
parameter.- Parameters:
value
- The input value.ctx
- AProcessFunction.Context
that allows querying the timestamp of the element 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.
-
onTimer
public void onTimer(long timestamp, ProcessFunction.OnTimerContext ctx, Collector<O> out) throws Exception
Called when a timer set usingTimerService
fires.- Parameters:
timestamp
- The timestamp of the firing timer.ctx
- AnProcessFunction.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.
-
-