[ 
https://issues.apache.org/jira/browse/FLINK-7974?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16247468#comment-16247468
 ] 

ASF GitHub Bot commented on FLINK-7974:
---------------------------------------

Github user tillrohrmann commented on a diff in the pull request:

    https://github.com/apache/flink/pull/4993#discussion_r150228589
  
    --- Diff: 
flink-queryable-state/flink-queryable-state-client-java/src/main/java/org/apache/flink/queryablestate/network/Client.java
 ---
    @@ -419,20 +440,27 @@ void close() {
                 * @param cause The cause to close the channel with.
                 * @return Channel close future
                 */
    -           private boolean close(Throwable cause) {
    +           private CompletableFuture<Boolean> close(Throwable cause) {
    +                   final CompletableFuture<Boolean> shutdownFuture = new 
CompletableFuture<>();
                        if (failureCause.compareAndSet(null, cause)) {
    -                           channel.close();
    -                           stats.reportInactiveConnection();
    +                           final CompletableFuture<?> tmp = new 
CompletableFuture<>();
    +                           channel.close().addListener(finished -> 
tmp.complete(null));
     
    -                           for (long requestId : pendingRequests.keySet()) 
{
    -                                   TimestampedCompletableFuture pending = 
pendingRequests.remove(requestId);
    -                                   if (pending != null && 
pending.completeExceptionally(cause)) {
    -                                           stats.reportFailedRequest();
    +                           tmp.thenRun(() -> {
    --- End diff --
    
    Instead of creating the additional indirection with `shutdownFuture`, 
couldn't we simply return the result here as teh shutdown future?


> AbstractServerBase#shutdown does not wait for shutdown completion
> -----------------------------------------------------------------
>
>                 Key: FLINK-7974
>                 URL: https://issues.apache.org/jira/browse/FLINK-7974
>             Project: Flink
>          Issue Type: Bug
>          Components: Queryable State
>    Affects Versions: 1.4.0
>            Reporter: Till Rohrmann
>            Assignee: Kostas Kloudas
>            Priority: Critical
>
> The {{AbstractServerBase}} does not wait for the completion of its shutdown 
> when calling {{AbstractServerBase#shutdown}}. This is problematic since it 
> leads to resource leaks and instable tests such as the 
> {{AbstractServerTest}}. I propose to let the {{AbstractServerBase#shutdown}} 
> return a termination future which is completed upon shutdown completion.



--
This message was sent by Atlassian JIRA
(v6.4.14#64029)

Reply via email to