You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pulsar.apache.org by sa...@apache.org on 2020/06/24 19:29:38 UTC

[pulsar] branch master updated: Fix race condition in which exitFuture in FunctionAssignmentTailer never gets completed even though the tailer thread has exited (#7351)

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

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


The following commit(s) were added to refs/heads/master by this push:
     new 14ac116  Fix race condition in which exitFuture in FunctionAssignmentTailer never gets completed even though the tailer thread has exited (#7351)
14ac116 is described below

commit 14ac1162d5e506fc0ff3625605f01e3b363c3972
Author: Boyang Jerry Peng <je...@gmail.com>
AuthorDate: Wed Jun 24 12:28:54 2020 -0700

    Fix race condition in which exitFuture in FunctionAssignmentTailer never gets completed even though the tailer thread has exited (#7351)
    
    * fix race condition in which exitFuture in FunctionAssignmentTailer never gets completed even though the tailer thread has exited
    
    * fix
    
    Co-authored-by: Jerry Peng <je...@splunk.com>
---
 .../apache/pulsar/functions/worker/FunctionAssignmentTailer.java  | 8 +++++---
 1 file changed, 5 insertions(+), 3 deletions(-)

diff --git a/pulsar-functions/worker/src/main/java/org/apache/pulsar/functions/worker/FunctionAssignmentTailer.java b/pulsar-functions/worker/src/main/java/org/apache/pulsar/functions/worker/FunctionAssignmentTailer.java
index e1d9c1f..d2ee952 100644
--- a/pulsar-functions/worker/src/main/java/org/apache/pulsar/functions/worker/FunctionAssignmentTailer.java
+++ b/pulsar-functions/worker/src/main/java/org/apache/pulsar/functions/worker/FunctionAssignmentTailer.java
@@ -80,7 +80,6 @@ public class FunctionAssignmentTailer implements AutoCloseable {
             if (tailerThread == null || !tailerThread.isAlive()) {
                 tailerThread = getTailerThread();
             }
-            exitFuture = new CompletableFuture<>();
             tailerThread.start();
         }
     }
@@ -113,15 +112,18 @@ public class FunctionAssignmentTailer implements AutoCloseable {
                     }
                 }
                 tailerThread = null;
+
+                // complete exit future to be safe
+                exitFuture.complete(null);
+                // reset the future
+                exitFuture = new CompletableFuture<>();
             }
             if (reader != null) {
                 reader.close();
                 reader = null;
             }
 
-            exitFuture = null;
             exitOnEndOfTopic = false;
-            
         } catch (IOException e) {
             log.error("Failed to stop function assignment tailer", e);
         }