You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by sr...@apache.org on 2018/04/17 12:02:14 UTC

flink git commit: [hotfix] Clear interrupted flag in stream task cancellation

Repository: flink
Updated Branches:
  refs/heads/release-1.4 a9b497749 -> a057cbf65


[hotfix] Clear interrupted flag in stream task cancellation

We clear the interrupted flag before the cleanup code block of task cancellation.
Otherwise, code that would like to wait until services are properly shutdown will
always immediately return from calls that are supposed to be blocking waits.

(cherry picked from commit f9a583b)

(cherry picked from commit 1687e7a)


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

Branch: refs/heads/release-1.4
Commit: a057cbf6517081e424c365061556405b5bbd19db
Parents: a9b4977
Author: Stefan Richter <s....@data-artisans.com>
Authored: Fri Feb 23 18:21:24 2018 +0100
Committer: Stefan Richter <s....@data-artisans.com>
Committed: Tue Apr 17 14:02:00 2018 +0200

----------------------------------------------------------------------
 .../java/org/apache/flink/streaming/runtime/tasks/StreamTask.java | 3 +++
 1 file changed, 3 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/a057cbf6/flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java
----------------------------------------------------------------------
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java
index 6ae45c6..e18ee18 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java
@@ -305,6 +305,9 @@ public abstract class StreamTask<OUT, OP extends StreamOperator<OUT>>
 			// clean up everything we initialized
 			isRunning = false;
 
+			// clear the interrupted status so that we can wait for the following resource shutdowns to complete
+			Thread.interrupted();
+
 			// stop all timers and threads
 			if (timerService != null && !timerService.isTerminated()) {
 				try {