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

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

commit 8d4be699f80eb6e7e771228098fa4bf6d68db1eb
Author: Piotr Nowojski <[email protected]>
AuthorDate: Wed Dec 30 15:04:37 2020 +0100

    [hotfix][task] Optimize and simplify Task#isBackPressured
---
 .../main/java/org/apache/flink/runtime/taskmanager/Task.java   | 10 +++++-----
 1 file changed, 5 insertions(+), 5 deletions(-)

diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/Task.java 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/Task.java
index 51088a2..7cf2d48 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/Task.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/Task.java
@@ -511,12 +511,12 @@ public class Task
         if (invokable == null || consumableNotifyingPartitionWriters.length == 
0 || !isRunning()) {
             return false;
         }
-        final CompletableFuture<?>[] outputFutures =
-                new 
CompletableFuture[consumableNotifyingPartitionWriters.length];
-        for (int i = 0; i < outputFutures.length; ++i) {
-            outputFutures[i] = 
consumableNotifyingPartitionWriters[i].getAvailableFuture();
+        for (int i = 0; i < consumableNotifyingPartitionWriters.length; ++i) {
+            if (!consumableNotifyingPartitionWriters[i].isAvailable()) {
+                return true;
+            }
         }
-        return !CompletableFuture.allOf(outputFutures).isDone();
+        return false;
     }
 
     // ------------------------------------------------------------------------

Reply via email to