You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by ar...@apache.org on 2021/02/09 18:23:27 UTC

[flink] branch master updated (fc995d3 -> ced0fdd)

This is an automated email from the ASF dual-hosted git repository.

arvid pushed a change to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git.


    from fc995d3  [FLINK-21206] Write savepoints in unified format from HeapStateBackend
     new 3d40007  [FLINK-20847][checkpointing] Remove unused argument from shutdown() in CompletedCheckpointStore
     new ced0fdd  [hotfix][checkpointing] Update javadoc for CompletedCheckpointStore.shutdown()

The 2 revisions listed above as "new" are entirely new to this
repository and will be described in separate emails.  The revisions
listed as "add" were already present in the repository and have only
been added to this reference.


Summary of changes:
 .../flink/runtime/checkpoint/CompletedCheckpointStore.java       | 4 ++--
 .../DeactivatedCheckpointCompletedCheckpointStore.java           | 6 +++---
 .../runtime/checkpoint/DefaultCompletedCheckpointStore.java      | 5 ++---
 .../runtime/checkpoint/EmbeddedCompletedCheckpointStore.java     | 3 +--
 .../runtime/checkpoint/StandaloneCompletedCheckpointStore.java   | 3 +--
 .../java/org/apache/flink/runtime/scheduler/SchedulerBase.java   | 7 +------
 .../runtime/checkpoint/CheckpointCoordinatorFailureTest.java     | 3 +--
 .../runtime/checkpoint/CheckpointCoordinatorRestoringTest.java   | 2 +-
 .../flink/runtime/checkpoint/CheckpointCoordinatorTest.java      | 5 ++---
 .../flink/runtime/checkpoint/CompletedCheckpointStoreTest.java   | 2 +-
 .../runtime/checkpoint/DefaultCompletedCheckpointStoreTest.java  | 4 ++--
 .../checkpoint/ExecutionGraphCheckpointCoordinatorTest.java      | 3 +--
 .../checkpoint/StandaloneCompletedCheckpointStoreTest.java       | 4 ++--
 .../checkpoint/ZooKeeperCompletedCheckpointStoreITCase.java      | 6 +++---
 .../checkpoint/ZooKeeperCompletedCheckpointStoreTest.java        | 9 ++-------
 15 files changed, 25 insertions(+), 41 deletions(-)


[flink] 01/02: [FLINK-20847][checkpointing] Remove unused argument from shutdown() in CompletedCheckpointStore

Posted by ar...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

arvid pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git

commit 3d400072f28ac4ff37bd38ff077bd95d57302658
Author: Roman Khachatryan <kh...@gmail.com>
AuthorDate: Sat Feb 6 13:31:02 2021 +0100

    [FLINK-20847][checkpointing] Remove unused argument from shutdown() in CompletedCheckpointStore
---
 .../flink/runtime/checkpoint/CompletedCheckpointStore.java       | 3 +--
 .../DeactivatedCheckpointCompletedCheckpointStore.java           | 6 +++---
 .../runtime/checkpoint/DefaultCompletedCheckpointStore.java      | 5 ++---
 .../runtime/checkpoint/EmbeddedCompletedCheckpointStore.java     | 3 +--
 .../runtime/checkpoint/StandaloneCompletedCheckpointStore.java   | 3 +--
 .../java/org/apache/flink/runtime/scheduler/SchedulerBase.java   | 7 +------
 .../runtime/checkpoint/CheckpointCoordinatorFailureTest.java     | 3 +--
 .../runtime/checkpoint/CheckpointCoordinatorRestoringTest.java   | 2 +-
 .../flink/runtime/checkpoint/CheckpointCoordinatorTest.java      | 5 ++---
 .../flink/runtime/checkpoint/CompletedCheckpointStoreTest.java   | 2 +-
 .../runtime/checkpoint/DefaultCompletedCheckpointStoreTest.java  | 4 ++--
 .../checkpoint/ExecutionGraphCheckpointCoordinatorTest.java      | 3 +--
 .../checkpoint/StandaloneCompletedCheckpointStoreTest.java       | 4 ++--
 .../checkpoint/ZooKeeperCompletedCheckpointStoreITCase.java      | 6 +++---
 .../checkpoint/ZooKeeperCompletedCheckpointStoreTest.java        | 9 ++-------
 15 files changed, 24 insertions(+), 41 deletions(-)

diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java
index 1ed2aa1..789fb9b 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java
@@ -93,8 +93,7 @@ public interface CompletedCheckpointStore {
      *
      * @param jobStatus Job state on shut down
      */
-    void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup)
-            throws Exception;
+    void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner) throws Exception;
 
     /**
      * Returns all {@link CompletedCheckpoint} instances.
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DeactivatedCheckpointCompletedCheckpointStore.java b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DeactivatedCheckpointCompletedCheckpointStore.java
index 1a51a89..f0ef3cf 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DeactivatedCheckpointCompletedCheckpointStore.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DeactivatedCheckpointCompletedCheckpointStore.java
@@ -24,7 +24,8 @@ import java.util.List;
 
 /**
  * This class represents a {@link CompletedCheckpointStore} if checkpointing has been enabled.
- * Consequently, no component should use methods other than {@link #shutdown}.
+ * Consequently, no component should use methods other than {@link
+ * CompletedCheckpointStore#shutdown}.
  */
 public enum DeactivatedCheckpointCompletedCheckpointStore implements CompletedCheckpointStore {
     INSTANCE;
@@ -44,8 +45,7 @@ public enum DeactivatedCheckpointCompletedCheckpointStore implements CompletedCh
     }
 
     @Override
-    public void shutdown(
-            JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup)
+    public void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner)
             throws Exception {}
 
     @Override
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStore.java b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStore.java
index 091f40d..0db2d07 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStore.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStore.java
@@ -259,8 +259,7 @@ public class DefaultCompletedCheckpointStore<R extends ResourceVersion<R>>
     }
 
     @Override
-    public void shutdown(
-            JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup)
+    public void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner)
             throws Exception {
         if (jobStatus.isGloballyTerminalState()) {
             LOG.info("Shutting down");
@@ -270,7 +269,7 @@ public class DefaultCompletedCheckpointStore<R extends ResourceVersion<R>>
                         checkpoint,
                         checkpoint.shouldBeDiscardedOnShutdown(jobStatus),
                         checkpointsCleaner,
-                        postCleanup);
+                        () -> {});
             }
 
             completedCheckpoints.clear();
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/EmbeddedCompletedCheckpointStore.java b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/EmbeddedCompletedCheckpointStore.java
index 1c75b6f..174f0ea 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/EmbeddedCompletedCheckpointStore.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/EmbeddedCompletedCheckpointStore.java
@@ -74,8 +74,7 @@ public class EmbeddedCompletedCheckpointStore implements CompletedCheckpointStor
     }
 
     @Override
-    public void shutdown(
-            JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup)
+    public void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner)
             throws Exception {
         if (jobStatus.isGloballyTerminalState()) {
             checkpoints.clear();
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStore.java b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStore.java
index bc163cd..5863ec9 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStore.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStore.java
@@ -96,8 +96,7 @@ public class StandaloneCompletedCheckpointStore implements CompletedCheckpointSt
     }
 
     @Override
-    public void shutdown(
-            JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup)
+    public void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner)
             throws Exception {
         try {
             LOG.info("Shutting down");
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java
index efe665e..0e83645 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java
@@ -246,12 +246,7 @@ public abstract class SchedulerBase implements SchedulerNG {
         Exception exception = null;
 
         try {
-            completedCheckpointStore.shutdown(
-                    jobStatus,
-                    checkpointsCleaner,
-                    () -> {
-                        // don't schedule anything on shutdown
-                    });
+            completedCheckpointStore.shutdown(jobStatus, checkpointsCleaner);
         } catch (Exception e) {
             exception = e;
         }
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorFailureTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorFailureTest.java
index ddc2aac..a267f8c 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorFailureTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorFailureTest.java
@@ -181,8 +181,7 @@ public class CheckpointCoordinatorFailureTest extends TestLogger {
         }
 
         @Override
-        public void shutdown(
-                JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup)
+        public void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner)
                 throws Exception {
             throw new UnsupportedOperationException("Not implemented.");
         }
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorRestoringTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorRestoringTest.java
index e8134cf..8fa53b8 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorRestoringTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorRestoringTest.java
@@ -200,7 +200,7 @@ public class CheckpointCoordinatorRestoringTest extends TestLogger {
         assertEquals(1, completedCheckpoints.size());
 
         // shutdown the store
-        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner(), () -> {});
+        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner());
 
         // restore the store
         Set<ExecutionJobVertex> tasks = new HashSet<>();
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorTest.java
index 5b12701..e526bc2 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorTest.java
@@ -1345,8 +1345,7 @@ public class CheckpointCoordinatorTest extends TestLogger {
             verify(subtaskState13, times(1)).discardState();
 
             checkpointCoordinator.shutdown();
-            completedCheckpointStore.shutdown(
-                    JobStatus.FINISHED, new CheckpointsCleaner(), () -> {});
+            completedCheckpointStore.shutdown(JobStatus.FINISHED, new CheckpointsCleaner());
 
             // validate that the states in the second checkpoint have been discarded
             verify(subtaskState21, times(1)).discardState();
@@ -2689,7 +2688,7 @@ public class CheckpointCoordinatorTest extends TestLogger {
         }
 
         // shutdown the store
-        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner(), () -> {});
+        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner());
 
         // restore the store
         Set<ExecutionJobVertex> tasks = new HashSet<>();
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStoreTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStoreTest.java
index af43765..6163323 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStoreTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStoreTest.java
@@ -183,7 +183,7 @@ public abstract class CompletedCheckpointStoreTest extends TestLogger {
             checkpoints.addCheckpoint(checkpoint, new CheckpointsCleaner(), () -> {});
         }
 
-        checkpoints.shutdown(JobStatus.FINISHED, new CheckpointsCleaner(), () -> {});
+        checkpoints.shutdown(JobStatus.FINISHED, new CheckpointsCleaner());
 
         // Empty state
         assertNull(checkpoints.getLatestCheckpoint(false));
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStoreTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStoreTest.java
index 58207af..0430085 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStoreTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/DefaultCompletedCheckpointStoreTest.java
@@ -241,7 +241,7 @@ public class DefaultCompletedCheckpointStoreTest extends TestLogger {
         completedCheckpointStore.recover();
         assertThat(completedCheckpointStore.getAllCheckpoints().size(), is(num));
 
-        completedCheckpointStore.shutdown(JobStatus.CANCELED, new CheckpointsCleaner(), () -> {});
+        completedCheckpointStore.shutdown(JobStatus.CANCELED, new CheckpointsCleaner());
         assertThat(removeCalledNum.get(), is(num));
         assertThat(clearEntriesAllFuture.isDone(), is(true));
         assertThat(completedCheckpointStore.getAllCheckpoints().size(), is(0));
@@ -269,7 +269,7 @@ public class DefaultCompletedCheckpointStoreTest extends TestLogger {
         completedCheckpointStore.recover();
         assertThat(completedCheckpointStore.getAllCheckpoints().size(), is(3));
 
-        completedCheckpointStore.shutdown(JobStatus.CANCELLING, new CheckpointsCleaner(), () -> {});
+        completedCheckpointStore.shutdown(JobStatus.CANCELLING, new CheckpointsCleaner());
         try {
             removeAllFuture.get(timeout, TimeUnit.MILLISECONDS);
             fail("We should get an expected timeout because the job is not globally terminated.");
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ExecutionGraphCheckpointCoordinatorTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ExecutionGraphCheckpointCoordinatorTest.java
index e9325fa..d458113 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ExecutionGraphCheckpointCoordinatorTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ExecutionGraphCheckpointCoordinatorTest.java
@@ -246,8 +246,7 @@ public class ExecutionGraphCheckpointCoordinatorTest extends TestLogger {
         }
 
         @Override
-        public void shutdown(
-                JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner, Runnable postCleanup) {
+        public void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner) {
             shutdownStatus.complete(jobStatus);
         }
 
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStoreTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStoreTest.java
index becc973..54fdb09 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStoreTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/StandaloneCompletedCheckpointStoreTest.java
@@ -62,7 +62,7 @@ public class StandaloneCompletedCheckpointStoreTest extends CompletedCheckpointS
         assertEquals(1, store.getNumberOfRetainedCheckpoints());
         verifyCheckpointRegistered(operatorStates, sharedStateRegistry);
 
-        store.shutdown(JobStatus.FINISHED, new CheckpointsCleaner(), () -> {});
+        store.shutdown(JobStatus.FINISHED, new CheckpointsCleaner());
         assertEquals(0, store.getNumberOfRetainedCheckpoints());
         assertTrue(checkpoint.isDiscarded());
         verifyCheckpointDiscarded(operatorStates);
@@ -83,7 +83,7 @@ public class StandaloneCompletedCheckpointStoreTest extends CompletedCheckpointS
         assertEquals(1, store.getNumberOfRetainedCheckpoints());
         verifyCheckpointRegistered(taskStates, sharedStateRegistry);
 
-        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner(), () -> {});
+        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner());
         assertEquals(0, store.getNumberOfRetainedCheckpoints());
         assertTrue(checkpoint.isDiscarded());
         verifyCheckpointDiscarded(taskStates);
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreITCase.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreITCase.java
index 7dc0822..ddb8ce4 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreITCase.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreITCase.java
@@ -169,7 +169,7 @@ public class ZooKeeperCompletedCheckpointStoreITCase extends CompletedCheckpoint
                                         + checkpointStoreUtil.checkpointIDToName(
                                                 checkpoint.getCheckpointID())));
 
-        store.shutdown(JobStatus.FINISHED, new CheckpointsCleaner(), () -> {});
+        store.shutdown(JobStatus.FINISHED, new CheckpointsCleaner());
         assertEquals(0, store.getNumberOfRetainedCheckpoints());
         assertNull(
                 client.checkExists()
@@ -205,7 +205,7 @@ public class ZooKeeperCompletedCheckpointStoreITCase extends CompletedCheckpoint
                                         + checkpointStoreUtil.checkpointIDToName(
                                                 checkpoint.getCheckpointID())));
 
-        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner(), () -> {});
+        store.shutdown(JobStatus.SUSPENDED, new CheckpointsCleaner());
 
         assertEquals(0, store.getNumberOfRetainedCheckpoints());
 
@@ -391,7 +391,7 @@ public class ZooKeeperCompletedCheckpointStoreITCase extends CompletedCheckpoint
                 checkpointRequestDecider
                         .chooseRequestToExecute(regularCheckpoint(), false, 0)
                         .isPresent());
-        checkpointStore.shutdown(JobStatus.FINISHED, checkpointsCleaner, () -> {});
+        checkpointStore.shutdown(JobStatus.FINISHED, checkpointsCleaner);
     }
 
     static class HeapRetrievableStateHandle<T extends Serializable>
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreTest.java
index 6b4e682..0811a7e 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/ZooKeeperCompletedCheckpointStoreTest.java
@@ -184,12 +184,7 @@ public class ZooKeeperCompletedCheckpointStoreTest extends TestLogger {
         try {
             test.accept(store, checkpointsInZk, sharedStateRegistry);
         } finally {
-            store.shutdown(
-                    JobStatus.FINISHED,
-                    new CheckpointsCleaner(),
-                    () -> {
-                        /* no op */
-                    });
+            store.shutdown(JobStatus.FINISHED, new CheckpointsCleaner());
             sharedStateRegistry.close();
         }
     }
@@ -254,7 +249,7 @@ public class ZooKeeperCompletedCheckpointStoreTest extends TestLogger {
             checkpointStore.addCheckpoint(checkpoint1, new CheckpointsCleaner(), () -> {});
             assertThat(checkpointStore.getAllCheckpoints(), Matchers.contains(checkpoint1));
 
-            checkpointStore.shutdown(JobStatus.FINISHED, new CheckpointsCleaner(), () -> {});
+            checkpointStore.shutdown(JobStatus.FINISHED, new CheckpointsCleaner());
 
             // verify that the checkpoint is discarded
             CompletedCheckpointStoreTest.verifyCheckpointDiscarded(checkpoint1);


[flink] 02/02: [hotfix][checkpointing] Update javadoc for CompletedCheckpointStore.shutdown()

Posted by ar...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

arvid pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git

commit ced0fdddc31114516b94c97ae021aceeb53bb991
Author: Roman Khachatryan <kh...@gmail.com>
AuthorDate: Sat Feb 6 13:34:12 2021 +0100

    [hotfix][checkpointing] Update javadoc for CompletedCheckpointStore.shutdown()
---
 .../org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java    | 1 +
 1 file changed, 1 insertion(+)

diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java
index 789fb9b..10c5c07 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CompletedCheckpointStore.java
@@ -92,6 +92,7 @@ public interface CompletedCheckpointStore {
      * or kept.
      *
      * @param jobStatus Job state on shut down
+     * @param checkpointsCleaner that will cleanup copmpleted checkpoints if needed
      */
     void shutdown(JobStatus jobStatus, CheckpointsCleaner checkpointsCleaner) throws Exception;