Skip to content

Commit

Permalink
[hotfix] Fix typos and other small things
Browse files Browse the repository at this point in the history
  • Loading branch information
bowenli86 authored and aljoscha committed Sep 25, 2017
1 parent cb615f1 commit d8ed58b
Show file tree
Hide file tree
Showing 4 changed files with 23 additions and 18 deletions.
24 changes: 13 additions & 11 deletions docs/dev/stream/operators/windows.md
Original file line number Diff line number Diff line change
Expand Up @@ -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.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -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 <T> The type of the input element.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -35,7 +35,7 @@
/**
* An example that illustrates the use of side outputs.
*
* <p>This is a modified version of {@link WindowWordCount}
* <p>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.
*/
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -79,10 +79,10 @@
* <p>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.
*
* <p>Note that the {@code AllWindowedStream} is purely and API construct, during runtime
* <p>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.
*
Expand All @@ -108,8 +108,7 @@ public class AllWindowedStream<T, W extends Window> {
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<T> lateDataOutputTag;

Expand Down Expand Up @@ -254,7 +253,11 @@ public <R> SingleOutputStreamOperator<R> reduce(
* @return The data stream that is the result of applying the window function to the window.
*/
@PublicEvolving
public <R> SingleOutputStreamOperator<R> reduce(ReduceFunction<T> reduceFunction, AllWindowFunction<T, R, W> function, TypeInformation<R> resultType) {
public <R> SingleOutputStreamOperator<R> reduce(
ReduceFunction<T> reduceFunction,
AllWindowFunction<T, R, W> function,
TypeInformation<R> resultType) {

if (reduceFunction instanceof RichFunction) {
throw new UnsupportedOperationException("ReduceFunction of reduce can not be a RichFunction.");
}
Expand Down

0 comments on commit d8ed58b

Please sign in to comment.