Repository: flink
Updated Branches:
  refs/heads/release-1.4 4b2178677 -> 74135c9db


[FLINK-8116] [DataStream] Fix stale comments referring to Checkpointed interface


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

Branch: refs/heads/release-1.4
Commit: 6884ce98f3936248a534e259789b9adde1ff9514
Parents: 4b21786
Author: Ankit Parashar <[email protected]>
Authored: Mon Dec 4 23:46:16 2017 +0530
Committer: Tzu-Li (Gordon) Tai <[email protected]>
Committed: Fri Jan 5 22:01:43 2018 -0800

----------------------------------------------------------------------
 docs/ops/state/state_backends.md                      |  2 +-
 .../api/functions/source/SourceFunction.java          | 14 +++++++-------
 .../api/scala/StreamExecutionEnvironment.scala        |  2 +-
 3 files changed, 9 insertions(+), 9 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/6884ce98/docs/ops/state/state_backends.md
----------------------------------------------------------------------
diff --git a/docs/ops/state/state_backends.md b/docs/ops/state/state_backends.md
index b32ad9f..cc2ffde 100644
--- a/docs/ops/state/state_backends.md
+++ b/docs/ops/state/state_backends.md
@@ -26,7 +26,7 @@ Programs written in the [Data Stream API]({{ site.baseurl 
}}/dev/datastream_api.
 
 - Windows gather elements or aggregates until they are triggered
 - Transformation functions may use the key/value state interface to store 
values
-- Transformation functions may implement the `Checkpointed` interface to make 
their local variables fault tolerant
+- Transformation functions may implement the `CheckpointedFunction` interface 
to make their local variables fault tolerant
 
 See also [state section]({{ site.baseurl }}/dev/stream/state/index.html) in 
the streaming API guide.
 

http://git-wip-us.apache.org/repos/asf/flink/blob/6884ce98/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java
----------------------------------------------------------------------
diff --git 
a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java
 
b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java
index 4665cc6..cb2e15f 100644
--- 
a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java
+++ 
b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java
@@ -34,9 +34,9 @@ import java.io.Serializable;
  * The run method can run for as long as necessary. The source must, however, 
react to an
  * invocation of {@link #cancel()} by breaking out of its main loop.
  *
- * <h3>Checkpointed Sources</h3>
+ * <h3>CheckpointedFunction Sources</h3>
  *
- * <p>Sources that also implement the {@link 
org.apache.flink.streaming.api.checkpoint.Checkpointed}
+ * <p>Sources that also implement the {@link 
org.apache.flink.streaming.api.checkpoint.CheckpointedFunction}
  * interface must ensure that state checkpointing, updating of internal state 
and emission of
  * elements are not done concurrently. This is achieved by using the provided 
checkpointing lock
  * object to protect update of state and emission of elements in a 
synchronized block.
@@ -44,7 +44,7 @@ import java.io.Serializable;
  * <p>This is the basic pattern one should follow when implementing a 
(checkpointed) source:
  *
  * <pre>{@code
- *  public class ExampleSource<T> implements SourceFunction<T>, 
Checkpointed<Long> {
+ *  public class ExampleSource<T> implements SourceFunction<T>, 
CheckpointedFunction {
  *      private long count = 0L;
  *      private volatile boolean isRunning = true;
  *
@@ -61,9 +61,9 @@ import java.io.Serializable;
  *          isRunning = false;
  *      }
  *
- *      public Long snapshotState(long checkpointId, long checkpointTimestamp) 
{ return count; }
+ *      public void snapshotState(FunctionSnapshotContext context) {  }
  *
- *      public void restoreState(Long state) { this.count = state; }
+ *      public void initializeState(FunctionInitializationContext context) {  }
  * }
  * }</pre>
  *
@@ -96,12 +96,12 @@ public interface SourceFunction<T> extends Function, 
Serializable {
         * Starts the source. Implementations can use the {@link SourceContext} 
emit
         * elements.
         *
-        * <p>Sources that implement {@link 
org.apache.flink.streaming.api.checkpoint.Checkpointed}
+        * <p>Sources that implement {@link 
org.apache.flink.streaming.api.checkpoint.CheckpointedFunction}
         * must lock on the checkpoint lock (using a synchronized block) before 
updating internal
         * state and emitting elements, to make both an atomic operation:
         *
         * <pre>{@code
-        *  public class ExampleSource<T> implements SourceFunction<T>, 
Checkpointed<Long> {
+        *  public class ExampleSource<T> implements SourceFunction<T>, 
CheckpointedFunction<Long> {
         *      private long count = 0L;
         *      private volatile boolean isRunning = true;
         *

http://git-wip-us.apache.org/repos/asf/flink/blob/6884ce98/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
----------------------------------------------------------------------
diff --git 
a/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
 
b/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
index 9fd03c3..3bba505 100644
--- 
a/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
+++ 
b/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
@@ -229,7 +229,7 @@ class StreamExecutionEnvironment(javaEnv: JavaEnv) {
    * [[KeyedStream]] is maintained (heap, managed memory, externally), and 
where state
    * snapshots/checkpoints are stored, both for the key/value state, and for 
checkpointed
    * functions (implementing the interface 
-   * [[org.apache.flink.streaming.api.checkpoint.Checkpointed]].
+   * [[org.apache.flink.streaming.api.checkpoint.CheckpointedFunction]].
    *
    * <p>The [[org.apache.flink.runtime.state.memory.MemoryStateBackend]] for 
example
    * maintains the state in heap memory, as objects. It is lightweight without 
extra 

Reply via email to