Class ProcessWindowFunction<IN,OUT,KEY,W extends Window>
- java.lang.Object
-
- org.apache.flink.api.common.functions.AbstractRichFunction
-
- org.apache.flink.streaming.api.functions.windowing.ProcessWindowFunction<IN,OUT,KEY,W>
-
- Type Parameters:
IN
- The type of the input value.OUT
- The type of the output value.KEY
- The type of the key.W
- The type ofWindow
that this window function can be applied on.
- All Implemented Interfaces:
Serializable
,Function
,RichFunction
- Direct Known Subclasses:
ReduceApplyProcessWindowFunction
@PublicEvolving public abstract class ProcessWindowFunction<IN,OUT,KEY,W extends Window> extends AbstractRichFunction
Base abstract class for functions that are evaluated over keyed (grouped) windows using a context for retrieving extra information.- See Also:
- Serialized Form
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description class
ProcessWindowFunction.Context
The context holding window metadata.
-
Constructor Summary
Constructors Constructor Description ProcessWindowFunction()
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
clear(ProcessWindowFunction.Context context)
Deletes any state in theContext
when the Window expires (the watermark passes itsmaxTimestamp
+allowedLateness
).abstract void
process(KEY key, ProcessWindowFunction.Context context, Iterable<IN> elements, Collector<OUT> out)
Evaluates the window and outputs none or several elements.-
Methods inherited from class org.apache.flink.api.common.functions.AbstractRichFunction
close, getIterationRuntimeContext, getRuntimeContext, open, setRuntimeContext
-
-
-
-
Method Detail
-
process
public abstract void process(KEY key, ProcessWindowFunction.Context context, Iterable<IN> elements, Collector<OUT> out) throws Exception
Evaluates the window and outputs none or several elements.- Parameters:
key
- The key for which this window is evaluated.context
- The context in which the window is being evaluated.elements
- The elements in the window being evaluated.out
- A collector for emitting elements.- Throws:
Exception
- The function may throw exceptions to fail the program and trigger recovery.
-
clear
public void clear(ProcessWindowFunction.Context context) throws Exception
Deletes any state in theContext
when the Window expires (the watermark passes itsmaxTimestamp
+allowedLateness
).- Parameters:
context
- The context to which the window is being evaluated- Throws:
Exception
- The function may throw exceptions to fail the program and trigger recovery.
-
-