[hotfix] Updates the AssignerWithPunctuatedWatermarks and the 
AssignerWithPeriodicWatermarks javadocs

This closes #1811


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/6593e482
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/6593e482
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/6593e482

Branch: refs/heads/master
Commit: 6593e482e2bb0a66121d495eadd409dbc9b15cd5
Parents: 5cb84f1
Author: kl0u <[email protected]>
Authored: Thu Mar 17 12:37:30 2016 +0100
Committer: Stephan Ewen <[email protected]>
Committed: Mon Apr 4 21:31:49 2016 +0200

----------------------------------------------------------------------
 .../AssignerWithPeriodicWatermarks.java         | 31 ++++++++++----------
 .../AssignerWithPunctuatedWatermarks.java       | 27 ++++++++++-------
 2 files changed, 32 insertions(+), 26 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/6593e482/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPeriodicWatermarks.java
----------------------------------------------------------------------
diff --git 
a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPeriodicWatermarks.java
 
b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPeriodicWatermarks.java
index 0c37896..38ee394 100644
--- 
a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPeriodicWatermarks.java
+++ 
b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPeriodicWatermarks.java
@@ -27,15 +27,16 @@ import org.apache.flink.streaming.api.watermark.Watermark;
  * These timestamps and watermarks are used by functions and operators that 
operate
  * on event time, for example event time windows.
  * 
- * <p>This class is used to generate watermarks in a periodical interval.
+ * <p>Use this class to generate watermarks in a periodical interval.
  * At most every {@code i} milliseconds (configured via
  * {@link ExecutionConfig#getAutoWatermarkInterval()}, the system will call the
  * {@link #getCurrentWatermark()} method to probe for the next watermark value.
- * The system will generate a new watermark, if the probed value is larger than
- * zero and larger than the previous watermark.
- * 
+ * The system will generate a new watermark, if the probed value is non-null
+ * and has a timestamp larger than that of the previous watermark (to preserve
+ * the contract of ascending watermarks).
+ *
  * <p>The system may call the {@link #getCurrentWatermark()} method less often 
than every
- * {@code i} milliseconds, of no new elements arrived since the last call to 
the
+ * {@code i} milliseconds, if no new elements arrived since the last call to 
the
  * method.
  *
  * <p>Timestamps and watermarks are defined as {@code longs} that represent the
@@ -51,24 +52,24 @@ public interface AssignerWithPeriodicWatermarks<T> extends 
TimestampAssigner<T>
 
        /**
         * Returns the current watermark. This method is periodically called by 
the
-        * system to retrieve the current watermark. The method may return null 
to
+        * system to retrieve the current watermark. The method may return 
{@code null} to
         * indicate that no new Watermark is available.
         * 
-        * <p>The returned watermark will be emitted only if it is non-null and 
larger
-        * than the previously emitted watermark. If the current watermark is 
still
+        * <p>The returned watermark will be emitted only if it is non-null and 
its timestamp
+        * is larger than that of the previously emitted watermark (to preserve 
the contract of
+        * ascending watermarks). If the current watermark is still
         * identical to the previous one, no progress in event time has 
happened since
-        * the previous call to this method.
-        * 
-        * <p>If this method returns a value that is smaller than the 
previously returned watermark,
-        * then the implementation does not properly handle the event stream 
timestamps.
-        * In that case, the returned watermark will not be emitted (to 
preserve the contract of
-        * ascending watermarks), and the violation will be logged and 
registered in the metrics.
-        *     
+        * the previous call to this method. If a null value is returned, or 
the timestamp
+        * of the returned watermark is smaller than that of the last emitted 
one, then no
+        * new watermark will be generated.
+        *
         * <p>The interval in which this method is called and Watermarks are 
generated
         * depends on {@link ExecutionConfig#getAutoWatermarkInterval()}.
         *
         * @see org.apache.flink.streaming.api.watermark.Watermark
         * @see ExecutionConfig#getAutoWatermarkInterval()
+        *
+        * @return {@code Null}, if no watermark should be emitted, or the next 
watermark to emit.
         */
        Watermark getCurrentWatermark();
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/6593e482/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPunctuatedWatermarks.java
----------------------------------------------------------------------
diff --git 
a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPunctuatedWatermarks.java
 
b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPunctuatedWatermarks.java
index 42579e3..48f29b2 100644
--- 
a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPunctuatedWatermarks.java
+++ 
b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/AssignerWithPunctuatedWatermarks.java
@@ -23,9 +23,14 @@ import org.apache.flink.streaming.api.watermark.Watermark;
 /**
  * The {@code AssignerWithPunctuatedWatermarks} assigns event time timestamps 
to elements,
  * and generates low watermarks that signal event time progress within the 
stream.
- * 
- * <p>Use these class if certain special elements act as markers that signify 
event time
+ * These timestamps and watermarks are used by functions and operators that 
operate
+ * on event time, for example event time windows.
+ *
+ * <p>Use this class if certain special elements act as markers that signify 
event time
  * progress, and when you want to emit watermarks specifically at certain 
events.
+ * The system will generate a new watermark, if the probed value is non-null
+ * and has a timestamp larger than that of the previous watermark (to preserve
+ * the contract of ascending watermarks).
  * 
  * <p>For use cases that should periodically emit watermarks based on element 
timestamps,
  * use the {@link AssignerWithPeriodicWatermarks} instead.
@@ -33,7 +38,7 @@ import org.apache.flink.streaming.api.watermark.Watermark;
  * <p>The following example illustrates how to use this timestamp extractor 
and watermark
  * generator. It assumes elements carry a timestamp that describes when they 
were created,
  * and that some elements carry a flag, marking them as the end of a sequence 
such that no
- * elements with smaller timestamps can come any more.
+ * elements with smaller timestamps can come anymore.
  * 
  * <pre>{@code
  * public class WatermarkOnFlagAssigner implements 
AssignerWithPunctuatedWatermarks<MyElement> {
@@ -61,14 +66,14 @@ public interface AssignerWithPunctuatedWatermarks<T> 
extends TimestampAssigner<T
        
        /**
         * Asks this implementation if it wants to emit a watermark. This 
method is called right after
-        * the {@link #extractTimestamp(Object, long)} method. If the method 
returns a positive
-        * value, a new watermark should be emitted. If a negative value is 
emitted, no new watermark
-        * will be generated.
-        * 
-        * <p>Note that whenever this method returns a positive value that is 
larger than the previous
-        * value, a new watermark is generated. Hence, the implementation has 
full control how often
-        * watermarks are generated.
-        * 
+        * the {@link #extractTimestamp(Object, long)} method.
+        *
+        * <p>The returned watermark will be emitted only if it is non-null and 
its timestamp
+        * is larger than that of the previously emitted watermark (to preserve 
the contract of
+        * ascending watermarks). If a null value is returned, or the timestamp 
of the returned
+        * watermark is smaller than that of the last emitted one, then no new 
watermark will
+        * be generated.
+        *
         * <p>For an example how to use this method, see the documentation of
         * {@link AssignerWithPunctuatedWatermarks this class}.
         *

Reply via email to