pnowojski commented on a change in pull request #12714:
URL: https://github.com/apache/flink/pull/12714#discussion_r442803046
##########
File path:
flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/Task.java
##########
@@ -819,6 +823,8 @@ else if (transitionState(current, ExecutionState.FAILED,
t)) {
}
finally {
try {
+ waitInvokableCancelCompletion();
Review comment:
Could you add a test for clean shutdown case as well? (to cover for the
bug in the previous version that I commented in the PR)
##########
File path:
flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java
##########
@@ -657,20 +657,25 @@ protected void cleanUpInvoke() throws Exception {
}
@Override
- public final void cancel() throws Exception {
+ public final CompletableFuture<Void> cancel() {
isRunning = false;
canceled = true;
// the "cancel task" call must come first, but the cancelables
must be
// closed no matter what
- try {
- cancelTask();
- }
- finally {
- mailboxProcessor.allActionsCompleted();
- cancelables.close();
+ return FutureUtils
+ .completedVoidFuture()
+ .thenRun(this::cancelTask)
Review comment:
> Leaving client code to deal with both method throwing an exception and
Future completed exceptionally isn't very nice IMO.
That would be an argument if not for the fact that exceptions are (and still
should be probably) handled via the caller anyway.
But I guess you are right. To avoid code duplication with try/catch block,
you would have to extract the `whenComplete(...)` content to a separate method.
This could be argued which is a better/easier to understand way, so if you want
let's leave it as it is.
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]