diff --git a/docs/dev/stream/operators/windows.md b/docs/dev/stream/operators/windows.md index 012d5313742bd..b825876e734e4 100644 --- a/docs/dev/stream/operators/windows.md +++ b/docs/dev/stream/operators/windows.md @@ -35,21 +35,23 @@ for the rest of the page. **Keyed Windows** stream - .keyBy(...) <- keyed versus non-keyed windows - .window(...) <- required: "assigner" - [.trigger(...)] <- optional: "trigger" (else default trigger) - [.evictor(...)] <- optional: "evictor" (else no evictor) - [.allowedLateness()] <- optional, else zero - .reduce/fold/apply() <- required: "function" + .keyBy(...) <- keyed versus non-keyed windows + .window(...) <- required: "assigner" + [.trigger(...)] <- optional: "trigger" (else default trigger) + [.evictor(...)] <- optional: "evictor" (else no evictor) + [.allowedLateness(...)] <- optional: "lateness" (else zero) + [.sideOutputLateData(...)] <- optional: "output tag" (else no side output for late data) + .reduce/fold/apply() <- required: "function" **Non-Keyed Windows** stream - .windowAll(...) <- required: "assigner" - [.trigger(...)] <- optional: "trigger" (else default trigger) - [.evictor(...)] <- optional: "evictor" (else no evictor) - [.allowedLateness()] <- optional, else zero - .reduce/fold/apply() <- required: "function" + .windowAll(...) <- required: "assigner" + [.trigger(...)] <- optional: "trigger" (else default trigger) + [.evictor(...)] <- optional: "evictor" (else no evictor) + [.allowedLateness(...)] <- optional: "lateness" (else zero) + [.sideOutputLateData(...)] <- optional: "output tag" (else no side output for late data) + .reduce/fold/apply() <- required: "function" In the above, the commands in square brackets ([...]) are optional. This reveals that Flink allows you to customize your windowing logic in many different ways so that it best fits your needs. diff --git a/flink-core/src/main/java/org/apache/flink/api/java/functions/NullByteKeySelector.java b/flink-core/src/main/java/org/apache/flink/api/java/functions/NullByteKeySelector.java index 4aa533dd532f8..f521cfe06c7a3 100644 --- a/flink-core/src/main/java/org/apache/flink/api/java/functions/NullByteKeySelector.java +++ b/flink-core/src/main/java/org/apache/flink/api/java/functions/NullByteKeySelector.java @@ -22,7 +22,7 @@ /** * Used as a dummy {@link KeySelector} to allow using keyed operators - * for non-keyed usecases. Essentially, it gives all incoming records + * for non-keyed use cases. Essentially, it gives all incoming records * the same key, which is a {@code (byte) 0} value. * * @param The type of the input element. diff --git a/flink-examples/flink-examples-streaming/src/main/java/org/apache/flink/streaming/examples/sideoutput/SideOutputExample.java b/flink-examples/flink-examples-streaming/src/main/java/org/apache/flink/streaming/examples/sideoutput/SideOutputExample.java index 0cb89303d164d..dca7389f391f5 100644 --- a/flink-examples/flink-examples-streaming/src/main/java/org/apache/flink/streaming/examples/sideoutput/SideOutputExample.java +++ b/flink-examples/flink-examples-streaming/src/main/java/org/apache/flink/streaming/examples/sideoutput/SideOutputExample.java @@ -35,7 +35,7 @@ /** * An example that illustrates the use of side outputs. * - *

This is a modified version of {@link WindowWordCount} + *

This is a modified version of {@link org.apache.flink.streaming.examples.windowing.WindowWordCount} * that has a filter in the tokenizer and only emits some words for counting * while emitting the other words to a side output. */ diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/AllWindowedStream.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/AllWindowedStream.java index b0f31e16df44d..7fb3822992a29 100644 --- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/AllWindowedStream.java +++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/AllWindowedStream.java @@ -79,10 +79,10 @@ *

If an {@link org.apache.flink.streaming.api.windowing.evictors.Evictor} is specified it will be * used to evict elements from the window after * evaluation was triggered by the {@code Trigger} but before the actual evaluation of the window. - * When using an evictor window performance will degrade significantly, since + * When using an evictor, window performance will degrade significantly, since * pre-aggregation of window results cannot be used. * - *

Note that the {@code AllWindowedStream} is purely and API construct, during runtime + *

Note that the {@code AllWindowedStream} is purely an API construct, during runtime * the {@code AllWindowedStream} will be collapsed together with the * operation over the window into one single operation. * @@ -108,8 +108,7 @@ public class AllWindowedStream { private long allowedLateness = 0L; /** - * Side output {@code OutputTag} for late data. If no tag is set late data will simply be - * dropped. + * Side output {@code OutputTag} for late data. If no tag is set late data will simply be dropped. */ private OutputTag lateDataOutputTag; @@ -254,7 +253,11 @@ public SingleOutputStreamOperator reduce( * @return The data stream that is the result of applying the window function to the window. */ @PublicEvolving - public SingleOutputStreamOperator reduce(ReduceFunction reduceFunction, AllWindowFunction function, TypeInformation resultType) { + public SingleOutputStreamOperator reduce( + ReduceFunction reduceFunction, + AllWindowFunction function, + TypeInformation resultType) { + if (reduceFunction instanceof RichFunction) { throw new UnsupportedOperationException("ReduceFunction of reduce can not be a RichFunction."); }