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

dwysakowicz pushed a commit to branch release-1.15
in repository https://gitbox.apache.org/repos/asf/flink.git

commit 1fe11253572a1b1d1224de638e313666da346f62
Author: Dawid Wysakowicz <[email protected]>
AuthorDate: Tue Apr 12 17:17:40 2022 +0200

    [FLINK-26977] Unify 
SavepointITCase#testStopWithSavepointFailingAfterSnapshotCreation for both 
schedulers
    
    This closes #19439
---
 .../flink/test/checkpointing/SavepointITCase.java  | 46 +++-------------------
 1 file changed, 6 insertions(+), 40 deletions(-)

diff --git 
a/flink-tests/src/test/java/org/apache/flink/test/checkpointing/SavepointITCase.java
 
b/flink-tests/src/test/java/org/apache/flink/test/checkpointing/SavepointITCase.java
index 92ddcdf757c..4be614c93a2 100644
--- 
a/flink-tests/src/test/java/org/apache/flink/test/checkpointing/SavepointITCase.java
+++ 
b/flink-tests/src/test/java/org/apache/flink/test/checkpointing/SavepointITCase.java
@@ -1018,22 +1018,12 @@ public class SavepointITCase extends TestLogger {
                 // 2. job failover triggered by SchedulerBase.stopWithSavepoint
                 0,
                 (jobId, actualException) -> {
-                    if (ClusterOptions.isAdaptiveSchedulerEnabled(new 
Configuration())) {
-                        return actualException
-                                .getMessage()
-                                .contains("Stop with savepoint operation could 
not be completed");
-                    } else {
-                        Optional<StopWithSavepointStoppingException> 
actualFlinkException =
-                                findThrowable(
-                                        actualException, 
StopWithSavepointStoppingException.class);
-                        return actualFlinkException
-                                .map(
-                                        e ->
-                                                e.getMessage()
-                                                        .startsWith(
-                                                                "A savepoint 
has been created at:"))
-                                .orElse(false);
-                    }
+                    Optional<StopWithSavepointStoppingException> 
actualFlinkException =
+                            findThrowable(
+                                    actualException, 
StopWithSavepointStoppingException.class);
+                    return actualFlinkException
+                            .map(e -> e.getMessage().startsWith("A savepoint 
has been created at:"))
+                            .orElse(false);
                 },
                 false);
     }
@@ -1088,30 +1078,6 @@ public class SavepointITCase extends TestLogger {
         }
     }
 
-    private static BiFunction<JobID, ExecutionException, Boolean>
-            assertAfterSnapshotCreationFailure() {
-        return (jobId, actualException) -> {
-            if (ClusterOptions.isAdaptiveSchedulerEnabled(new 
Configuration())) {
-                return actualException
-                        .getMessage()
-                        .contains("Stop with savepoint operation could not be 
completed");
-            } else {
-                Optional<FlinkException> actualFlinkException =
-                        findThrowable(actualException, FlinkException.class);
-                if (!actualFlinkException.isPresent()) {
-                    return false;
-                }
-                return actualFlinkException
-                        .get()
-                        .getMessage()
-                        .contains(
-                                String.format(
-                                        "A global fail-over is triggered to 
recover the job %s.",
-                                        jobId));
-            }
-        };
-    }
-
     private static BiFunction<JobID, ExecutionException, Boolean>
             assertInSnapshotCreationFailure() {
         return (ignored, actualException) -> {

Reply via email to