Repository: storm
Updated Branches:
  refs/heads/1.0.x-branch 569cdfa5e -> 52875c24c


STORM-1750: Ensure worker dies when report-error-and-die is called. Make 
zookeeper_state_factory set-data try setting data if node creation fails 
because the node exists


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

Branch: refs/heads/1.0.x-branch
Commit: 6e4bdbc8cacd7621c570126a5d9e8da9f599d037
Parents: 0287d9b
Author: Stig Rohde Døssing <[email protected]>
Authored: Tue May 3 13:51:13 2016 +0200
Committer: Stig Rohde Døssing <[email protected]>
Committed: Tue May 3 13:51:13 2016 +0200

----------------------------------------------------------------------
 .../apache/storm/cluster_state/zookeeper_state_factory.clj  | 9 +++++++--
 storm-core/src/clj/org/apache/storm/daemon/executor.clj     | 5 ++++-
 2 files changed, 11 insertions(+), 3 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/storm/blob/6e4bdbc8/storm-core/src/clj/org/apache/storm/cluster_state/zookeeper_state_factory.clj
----------------------------------------------------------------------
diff --git 
a/storm-core/src/clj/org/apache/storm/cluster_state/zookeeper_state_factory.clj 
b/storm-core/src/clj/org/apache/storm/cluster_state/zookeeper_state_factory.clj
index 7eca328..06557d4 100644
--- 
a/storm-core/src/clj/org/apache/storm/cluster_state/zookeeper_state_factory.clj
+++ 
b/storm-core/src/clj/org/apache/storm/cluster_state/zookeeper_state_factory.clj
@@ -16,7 +16,7 @@
 
 (ns org.apache.storm.cluster-state.zookeeper-state-factory
   (:import [org.apache.curator.framework.state ConnectionStateListener])
-  (:import [org.apache.zookeeper KeeperException$NoNodeException]
+  (:import [org.apache.zookeeper KeeperException$NoNodeException 
KeeperException$NodeExistsException]
            [org.apache.storm.cluster ClusterState DaemonType])
   (:import [org.apache.storm.utils StormConnectionStateConverter])
   (:use [org.apache.storm cluster config log util])
@@ -92,7 +92,12 @@
          (zk/set-data zk-writer path data)
          (do
            (zk/mkdirs zk-writer (parent-path path) acls)
-           (zk/create-node zk-writer path data :persistent acls))))
+           (try
+             (zk/create-node zk-writer path data :persistent acls)
+             (catch RuntimeException e
+               (if (instance? KeeperException$NodeExistsException (.getCause 
e))
+                 (zk/set-data zk-writer path data)
+                 (throw e)))))))
 
      (set-worker-hb
        [this path data acls]

http://git-wip-us.apache.org/repos/asf/storm/blob/6e4bdbc8/storm-core/src/clj/org/apache/storm/daemon/executor.clj
----------------------------------------------------------------------
diff --git a/storm-core/src/clj/org/apache/storm/daemon/executor.clj 
b/storm-core/src/clj/org/apache/storm/daemon/executor.clj
index 07925b8..8c835d5 100644
--- a/storm-core/src/clj/org/apache/storm/daemon/executor.clj
+++ b/storm-core/src/clj/org/apache/storm/daemon/executor.clj
@@ -263,7 +263,10 @@
      :stream->component->grouper (outbound-components worker-context 
component-id storm-conf)
      :report-error (throttled-report-error-fn <>)
      :report-error-and-die (fn [error]
-                             ((:report-error <>) error)
+                             (try
+                               ((:report-error <>) error)
+                               (catch Exception e
+                                 (log-message "Error while reporting error to 
cluster, proceeding with shutdown")))
                              (if (or
                                     (exception-cause? InterruptedException 
error)
                                     (exception-cause? 
java.io.InterruptedIOException error))

Reply via email to