[27/45] curator git commit: Instead of closing the ZK instance and setting LOST directly, use the new testing API in 3.5 injectSessionExpiration(). This causes the ZK instance to close and LOST to get
Instead of closing the ZK instance and setting LOST directly, use the new testing API in 3.5 injectSessionExpiration(). This causes the ZK instance to close and LOST to get posted but has the benefit of sending a session expiration to all watchers Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/81bab455 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/81bab455 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/81bab455 Branch: refs/heads/CURATOR-248 Commit: 81bab455cf259f3b45d1fa0f7e26d78127f06d61 Parents: dd78816 Author: randgalt Authored: Mon Aug 24 00:21:35 2015 -0500 Committer: randgalt Committed: Mon Aug 24 00:21:35 2015 -0500 -- .../framework/state/ConnectionStateManager.java | 10 --- .../imps/TestEnabledSessionExpiredState.java| 28 2 files changed, 34 insertions(+), 4 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/81bab455/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index de71264..daa33f6 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -296,16 +296,18 @@ public class ConnectionStateManager implements Closeable int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; if ( elapsedMs >= useSessionTimeoutMs ) { -log.warn(String.format("Session timeout has elapsed while SUSPENDED. Posting LOST event and resetting the connection. Elapsed ms: %d. Session Timeout ms: %d", elapsedMs, useSessionTimeoutMs)); +log.warn(String.format("Session timeout has elapsed while SUSPENDED. Injecting a session expiration. Elapsed ms: %d. Session Timeout ms: %d", elapsedMs, useSessionTimeoutMs)); try { -client.getZookeeperClient().reset(); +// LOL - this method was proposed by me (JZ) in 2013 for totally unrelated reasons +// it got added to ZK 3.5 and now does exactly what we need +// https://issues.apache.org/jira/browse/ZOOKEEPER-1730 + client.getZookeeperClient().getZooKeeper().getTestable().injectSessionExpiration(); } catch ( Exception e ) { -log.error("Could not reset the connection", e); +log.error("Could not inject session expiration", e); } -addStateChange(ConnectionState.LOST); } } } http://git-wip-us.apache.org/repos/asf/curator/blob/81bab455/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java index a41d581..eff899d 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java @@ -29,11 +29,14 @@ import org.apache.curator.test.KillSession; import org.apache.curator.test.Timing; import org.apache.curator.utils.CloseableUtils; import org.apache.zookeeper.KeeperException; +import org.apache.zookeeper.WatchedEvent; +import org.apache.zookeeper.Watcher; import org.testng.Assert; import org.testng.annotations.AfterMethod; import org.testng.annotations.BeforeMethod; import org.testng.annotations.Test; import java.util.concurrent.BlockingQueue; +import java.util.concurrent.CountDownLatch; import java.util.concurrent.TimeUnit; public class TestEnabledSessionExpiredState extends BaseClassForTests @@ -79,6 +82,31 @@ public class TestEnabledSessionExpiredState extends BaseClassForTests } @Test +public void testInjectedWatchedEvent() throws Exception +{ +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED); + +final CountDownLatch latch = new CountDownLatch(1); +Watcher watcher = new Watcher() +{ +@Override +
[29/45] curator git commit: updated doc
updated doc Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/b8240a1d Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/b8240a1d Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/b8240a1d Branch: refs/heads/CURATOR-248 Commit: b8240a1dfcaf291196dc68fbfd1f2204c50951fc Parents: 1bbf301 Author: randgalt Authored: Mon Aug 24 09:28:53 2015 -0500 Committer: randgalt Committed: Mon Aug 24 09:28:53 2015 -0500 -- .../curator/framework/CuratorFrameworkFactory.java | 2 +- .../apache/curator/framework/state/ConnectionState.java | 7 --- .../framework/imps/TestEnabledSessionExpiredState.java | 11 +++ src/site/confluence/errors.confluence| 6 +++--- 4 files changed, 19 insertions(+), 7 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/b8240a1d/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index c59d78f..daffa13 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@ -381,7 +381,7 @@ public class CuratorFrameworkFactory * action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper * connection. Now, Curator will set the LOST state when any of the following occurs: * a) ZooKeeper returns a {@link Watcher.Event.KeeperState#Expired} or {@link KeeperException.Code#SESSIONEXPIRED}; - * b) Curator closes the internally managed ZooKeeper instance; c) The configured session timeout + * b) Curator closes the internally managed ZooKeeper instance; c) The session timeout * elapses during a network partition. * * http://git-wip-us.apache.org/repos/asf/curator/blob/b8240a1d/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java index fe40abf..f9f245a 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java @@ -18,6 +18,7 @@ */ package org.apache.curator.framework.state; +import org.apache.curator.connection.ConnectionHandlingPolicy; import org.apache.curator.framework.CuratorFrameworkFactory; import org.apache.zookeeper.KeeperException; import org.apache.zookeeper.Watcher; @@ -64,17 +65,17 @@ public enum ConnectionState /** * - * Curator will set the LOST state only when it believes that the ZooKeeper session + * Curator will set the LOST state when it believes that the ZooKeeper session * has expired. ZooKeeper connections have a session. When the session expires, clients must take appropriate * action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper * connection. Curator will set the LOST state when any of the following occurs: * a) ZooKeeper returns a {@link Watcher.Event.KeeperState#Expired} or {@link KeeperException.Code#SESSIONEXPIRED}; - * b) Curator closes the internally managed ZooKeeper instance; c) The configured session timeout + * b) Curator closes the internally managed ZooKeeper instance; c) The session timeout * elapses during a network partition. * * * - * NOTE: see {@link CuratorFrameworkFactory.Builder#connectionHandlingPolicy} for an important note about a + * NOTE: see {@link CuratorFrameworkFactory.Builder#connectionHandlingPolicy(ConnectionHandlingPolicy)} for an important note about a * change in meaning to LOST since 3.0.0 * */ http://git-wip-us.apache.org/repos/asf/curator/blob/b8240a1d/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabl
curator git commit: refined the tests
Repository: curator Updated Branches: refs/heads/CURATOR-248 c117b0853 -> 49b267d18 refined the tests Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/49b267d1 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/49b267d1 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/49b267d1 Branch: refs/heads/CURATOR-248 Commit: 49b267d185bddb7d2be073731ebc6a8352ed6efb Parents: c117b08 Author: randgalt Authored: Tue Sep 1 06:32:52 2015 -0700 Committer: randgalt Committed: Tue Sep 1 06:32:52 2015 -0700 -- .../framework/recipes/leader/TestLeaderSelector.java | 13 + 1 file changed, 9 insertions(+), 4 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/49b267d1/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelector.java -- diff --git a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelector.java b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelector.java index e5b9717..cd76bc1 100644 --- a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelector.java +++ b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelector.java @@ -37,6 +37,7 @@ import org.apache.curator.utils.CloseableUtils; import org.testng.Assert; import org.testng.annotations.Test; import org.testng.internal.annotations.Sets; +import java.util.Arrays; import java.util.List; import java.util.Set; import java.util.concurrent.ArrayBlockingQueue; @@ -103,8 +104,10 @@ public class TestLeaderSelector extends BaseClassForTests Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED.name()); Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), "leader"); server.close(); - Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.SUSPENDED.name()); - Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), "release"); +List next = Lists.newArrayList(); +next.add(changes.poll(timing.forSessionSleep().milliseconds(), TimeUnit.MILLISECONDS)); +next.add(changes.poll(timing.forSessionSleep().milliseconds(), TimeUnit.MILLISECONDS)); + Assert.assertTrue(next.equals(Arrays.asList(ConnectionState.SUSPENDED.name(), "release")) || next.equals(Arrays.asList("release", ConnectionState.SUSPENDED.name())), next.toString()); Assert.assertEquals(changes.poll(timing.forSessionSleep().milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.LOST.name()); selector.close(); @@ -130,8 +133,10 @@ public class TestLeaderSelector extends BaseClassForTests Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), "leader"); server.stop(); Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.SUSPENDED.name()); - Assert.assertEquals(changes.poll(timing.forSessionSleep().milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.LOST.name()); - Assert.assertEquals(changes.poll(timing.forWaiting().milliseconds(), TimeUnit.MILLISECONDS), "release"); +next = Lists.newArrayList(); +next.add(changes.poll(timing.forSessionSleep().milliseconds(), TimeUnit.MILLISECONDS)); +next.add(changes.poll(timing.forSessionSleep().milliseconds(), TimeUnit.MILLISECONDS)); + Assert.assertTrue(next.equals(Arrays.asList(ConnectionState.LOST.name(), "release")) || next.equals(Arrays.asList("release", ConnectionState.LOST.name())), next.toString()); } finally {
[38/45] curator git commit: return false if interrupted
return false if interrupted Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/38011678 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/38011678 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/38011678 Branch: refs/heads/CURATOR-248 Commit: 38011678c4faebb1cd9388e56f0d13d29862c609 Parents: 35b2063 Author: randgalt Authored: Mon Aug 31 07:19:21 2015 -0700 Committer: randgalt Committed: Mon Aug 31 07:19:21 2015 -0700 -- .../src/main/java/org/apache/curator/retry/RetryForever.java | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/38011678/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java -- diff --git a/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java b/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java index 89f0276..59f3d0d 100644 --- a/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java +++ b/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java @@ -51,8 +51,9 @@ public class RetryForever implements RetryPolicy } catch (InterruptedException e) { -log.warn("Error occurred while sleeping", e); Thread.currentThread().interrupt(); +log.warn("Error occurred while sleeping", e); +return false; } return true; }
[12/45] curator git commit: CURATOR-214 - Implemented new ZK API to allow Stat objects to be returned as part of the creation of a new ZNode.
CURATOR-214 - Implemented new ZK API to allow Stat objects to be returned as part of the creation of a new ZNode. Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/af369dec Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/af369dec Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/af369dec Branch: refs/heads/CURATOR-248 Commit: af369dec7a3ba20c60c68b502198bd851e34906f Parents: 7d97259 Author: Cam McKenzie Authored: Sun Aug 23 09:28:49 2015 +1000 Committer: Cam McKenzie Committed: Sun Aug 23 09:28:49 2015 +1000 -- ...reateModeStatBackgroundPathAndBytesable.java | 25 ++ .../api/CreateBackgroundModeStatACLable.java| 70 + .../curator/framework/api/CreateBuilder.java| 9 +- ...ateProtectACLCreateModePathAndBytesable.java | 72 + .../framework/imps/CreateBuilderImpl.java | 299 ++- .../framework/imps/TestCreateReturningStat.java | 199 .../framework/imps/TestFrameworkEdges.java | 4 +- 7 files changed, 658 insertions(+), 20 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/af369dec/curator-framework/src/main/java/org/apache/curator/framework/api/ACLCreateModeStatBackgroundPathAndBytesable.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/ACLCreateModeStatBackgroundPathAndBytesable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/ACLCreateModeStatBackgroundPathAndBytesable.java new file mode 100755 index 000..0becfb9 --- /dev/null +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/ACLCreateModeStatBackgroundPathAndBytesable.java @@ -0,0 +1,25 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.curator.framework.api; + +public interface ACLCreateModeStatBackgroundPathAndBytesable extends +ACLCreateModeBackgroundPathAndBytesable, +Statable> +{ +} http://git-wip-us.apache.org/repos/asf/curator/blob/af369dec/curator-framework/src/main/java/org/apache/curator/framework/api/CreateBackgroundModeStatACLable.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/CreateBackgroundModeStatACLable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/CreateBackgroundModeStatACLable.java new file mode 100755 index 000..2ebe63e --- /dev/null +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/CreateBackgroundModeStatACLable.java @@ -0,0 +1,70 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.curator.framework.api; + +import org.apache.zookeeper.CreateMode; + +public interface CreateBackgroundModeStatACLable extends +BackgroundPathAndBytesable, +CreateModable>, +ACLCreateModeBackgroundPathAndBytesable, +Statable +{ +/** + * Causes any parent nodes to get created if they haven't already been + * + * @return this + */ +public ACLCreateModePathAndBytesablecreatingParentsIfNeeded(); + +/** + * Causes any parent nodes to get created using {@link CreateMode#CONTAINER} if they haven't already been. +
[19/45] curator git commit: CURATOR-214 - Missed file.
CURATOR-214 - Missed file. Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/6c5a154c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/6c5a154c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/6c5a154c Branch: refs/heads/CURATOR-248 Commit: 6c5a154c8cb33bc0fa62f5ad9daac8d45db50262 Parents: ba4b5d8 Author: Cam McKenzie Authored: Sun Aug 23 14:40:12 2015 +1000 Committer: Cam McKenzie Committed: Sun Aug 23 14:40:12 2015 +1000 -- ...rotectACLCreateModeStatPathAndBytesable.java | 25 1 file changed, 25 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/6c5a154c/curator-framework/src/main/java/org/apache/curator/framework/api/ProtectACLCreateModeStatPathAndBytesable.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/ProtectACLCreateModeStatPathAndBytesable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/ProtectACLCreateModeStatPathAndBytesable.java new file mode 100644 index 000..07638d5 --- /dev/null +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/ProtectACLCreateModeStatPathAndBytesable.java @@ -0,0 +1,25 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.curator.framework.api; + +public interface ProtectACLCreateModeStatPathAndBytesable extends +ProtectACLCreateModePathAndBytesable, +Statable> +{ +}
[42/45] curator git commit: Use the negotiated session timeout in checkTimeouts()
Use the negotiated session timeout in checkTimeouts() Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/667b1d2c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/667b1d2c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/667b1d2c Branch: refs/heads/CURATOR-248 Commit: 667b1d2c7bd4b267585ae973dfff0f8b49f6b6ba Parents: 72f7223 Author: randgalt Authored: Mon Aug 31 19:48:12 2015 -0700 Committer: randgalt Committed: Mon Aug 31 19:48:12 2015 -0700 -- .../src/main/java/org/apache/curator/ConnectionState.java | 6 -- .../apache/curator/connection/ConnectionHandlingPolicy.java| 2 +- 2 files changed, 5 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/667b1d2c/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index f0996d8..eea2ce0 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -207,7 +207,9 @@ class ConnectionState implements Watcher, Closeable return zooKeeper.hasNewConnectionString(); } }; -ConnectionHandlingPolicy.CheckTimeoutsResult result = connectionHandlingPolicy.checkTimeouts(hasNewConnectionString, connectionStartMs, sessionTimeoutMs, connectionTimeoutMs); +int lastNegotiatedSessionTimeoutMs = getLastNegotiatedSessionTimeoutMs(); +int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; +ConnectionHandlingPolicy.CheckTimeoutsResult result = connectionHandlingPolicy.checkTimeouts(hasNewConnectionString, connectionStartMs, useSessionTimeoutMs, connectionTimeoutMs); switch ( result ) { default: @@ -227,7 +229,7 @@ class ConnectionState implements Watcher, Closeable if ( !Boolean.getBoolean(DebugUtils.PROPERTY_DONT_LOG_CONNECTION_ISSUES) ) { long elapsed = System.currentTimeMillis() - connectionStartMs; -int maxTimeout = Math.max(sessionTimeoutMs, connectionTimeoutMs); +int maxTimeout = Math.max(useSessionTimeoutMs, connectionTimeoutMs); log.warn(String.format("Connection attempt unsuccessful after %d (greater than max timeout of %d). Resetting connection and trying again with a new connection.", elapsed, maxTimeout)); } reset(); http://git-wip-us.apache.org/repos/asf/curator/blob/667b1d2c/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java index 6cea67d..9b6b895 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java @@ -63,7 +63,7 @@ public interface ConnectionHandlingPolicy * @param hasNewConnectionString proc to call to check if there is a new connection string. Important: the internal state is cleared after * this is called so you MUST handle the new connection string if true is returned * @param connectionStartMs the epoch/ms time that the connection was first initiated - * @param sessionTimeoutMs the configured session timeout in milliseconds + * @param sessionTimeoutMs the configured/negotiated session timeout in milliseconds * @param connectionTimeoutMs the configured connection timeout in milliseconds * @return result * @throws Exception errors
[14/45] curator git commit: major refactoring. Abstracting old/new behavior into a pluggable ConnectionHandlingPolicy. Also, IMPORTANT, made the new behavior the default. This needs to be discussed bu
major refactoring. Abstracting old/new behavior into a pluggable ConnectionHandlingPolicy. Also, IMPORTANT, made the new behavior the default. This needs to be discussed but it's a major improvement and we should default to it. Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/e2391370 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/e2391370 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/e2391370 Branch: refs/heads/CURATOR-248 Commit: e239137019608f02cabb23c27ab13adcef88c027 Parents: 6381ccb Author: randgalt Authored: Sat Aug 22 19:06:55 2015 -0500 Committer: randgalt Committed: Sat Aug 22 19:06:55 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 85 .../apache/curator/CuratorZookeeperClient.java | 32 .../main/java/org/apache/curator/RetryLoop.java | 28 +-- .../ClassicConnectionHandlingPolicy.java| 48 +++ .../connection/ConnectionHandlingPolicy.java| 84 +++ .../StandardConnectionHandlingPolicy.java | 35 .../java/org/apache/curator/TestEnsurePath.java | 5 +- .../framework/CuratorFrameworkFactory.java | 54 +++-- .../framework/imps/CuratorFrameworkImpl.java| 43 +- .../framework/state/ConnectionState.java| 20 + .../framework/state/ConnectionStateManager.java | 9 +-- .../imps/TestEnabledSessionExpiredState.java| 5 +- .../apache/curator/test/BaseClassForTests.java | 6 +- 13 files changed, 336 insertions(+), 118 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/e2391370/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index c3d6921..d6ddd33 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -18,9 +18,10 @@ */ package org.apache.curator; -import org.apache.curator.utils.CloseableUtils; +import org.apache.curator.connection.ConnectionHandlingPolicy; import org.apache.curator.drivers.TracerDriver; import org.apache.curator.ensemble.EnsembleProvider; +import org.apache.curator.utils.CloseableUtils; import org.apache.curator.utils.DebugUtils; import org.apache.curator.utils.ZookeeperFactory; import org.apache.zookeeper.KeeperException; @@ -32,6 +33,7 @@ import org.slf4j.LoggerFactory; import java.io.Closeable; import java.io.IOException; import java.util.Queue; +import java.util.concurrent.Callable; import java.util.concurrent.ConcurrentLinkedQueue; import java.util.concurrent.atomic.AtomicBoolean; import java.util.concurrent.atomic.AtomicLong; @@ -48,18 +50,19 @@ class ConnectionState implements Watcher, Closeable private final int sessionTimeoutMs; private final int connectionTimeoutMs; private final AtomicReference tracer; +private final ConnectionHandlingPolicy connectionHandlingPolicy; private final Queue backgroundExceptions = new ConcurrentLinkedQueue(); private final Queue parentWatchers = new ConcurrentLinkedQueue(); private final AtomicLong instanceIndex = new AtomicLong(); private volatile long connectionStartMs = 0; -private final AtomicBoolean enableTimeoutChecks = new AtomicBoolean(true); -ConnectionState(ZookeeperFactory zookeeperFactory, EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher parentWatcher, AtomicReference tracer, boolean canBeReadOnly) +ConnectionState(ZookeeperFactory zookeeperFactory, EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher parentWatcher, AtomicReference tracer, boolean canBeReadOnly, ConnectionHandlingPolicy connectionHandlingPolicy) { this.ensembleProvider = ensembleProvider; this.sessionTimeoutMs = sessionTimeoutMs; this.connectionTimeoutMs = connectionTimeoutMs; this.tracer = tracer; +this.connectionHandlingPolicy = connectionHandlingPolicy; if ( parentWatcher != null ) { parentWatchers.offer(parentWatcher); @@ -68,11 +71,6 @@ class ConnectionState implements Watcher, Closeable zooKeeper = new HandleHolder(zookeeperFactory, this, ensembleProvider, sessionTimeoutMs, canBeReadOnly); } -void disableTimeoutChecks() -{ -enableTimeoutChecks.set(false); -} - ZooKeeper getZooKeeper() throws Exception { if ( SessionFailRetryLoop.sessionForThreadHasFailed() ) @@ -87,13 +85,10 @@ class ConnectionState implements Watcher, Closeable
[28/45] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-247
Merge branch 'CURATOR-3.0' into CURATOR-247 Conflicts: curator-client/src/main/java/org/apache/curator/ConnectionState.java Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/1bbf301a Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/1bbf301a Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/1bbf301a Branch: refs/heads/CURATOR-248 Commit: 1bbf301ab75debf8866c78f13a32e65049a20772 Parents: 81bab45 f9af0ce Author: randgalt Authored: Mon Aug 24 08:02:08 2015 -0500 Committer: randgalt Committed: Mon Aug 24 08:02:08 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 36 +--- .../framework/imps/TestBlockUntilConnected.java | 24 + 2 files changed, 41 insertions(+), 19 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/1bbf301a/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --cc curator-client/src/main/java/org/apache/curator/ConnectionState.java index 555a52d,46ae9fd..f0996d8 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@@ -44,10 -41,9 +44,10 @@@ class ConnectionState implements Watche { private static final int MAX_BACKGROUND_EXCEPTIONS = 10; private static final boolean LOG_EVENTS = Boolean.getBoolean(DebugUtils.PROPERTY_LOG_EVENTS); - private final Logger log = LoggerFactory.getLogger(getClass()); + private static final Logger log = LoggerFactory.getLogger(ConnectionState.class); private final HandleHolder zooKeeper; private final AtomicBoolean isConnected = new AtomicBoolean(false); +private final AtomicInteger lastNegotiatedSessionTimeoutMs = new AtomicInteger(0); private final EnsembleProvider ensembleProvider; private final int sessionTimeoutMs; private final int connectionTimeoutMs; @@@ -156,29 -145,22 +156,27 @@@ log.debug("ConnectState watcher: " + event); } - for ( Watcher parentWatcher : parentWatchers ) - { - TimeTrace timeTrace = new TimeTrace("connection-state-parent-process", tracer.get()); - parentWatcher.process(event); - timeTrace.commit(); - } - - boolean wasConnected = isConnected.get(); - boolean newIsConnected = wasConnected; if ( event.getType() == Watcher.Event.EventType.None ) { - newIsConnected = checkState(event.getState(), wasConnected); + boolean wasConnected = isConnected.get(); + boolean newIsConnected = checkState(event.getState(), wasConnected); + if ( newIsConnected != wasConnected ) + { + isConnected.set(newIsConnected); + connectionStartMs = System.currentTimeMillis(); ++if ( newIsConnected ) ++{ ++ lastNegotiatedSessionTimeoutMs.set(zooKeeper.getNegotiatedSessionTimeoutMs()); ++log.debug("Negotiated session timeout: " + lastNegotiatedSessionTimeoutMs.get()); ++} + } } - if ( newIsConnected != wasConnected ) + for ( Watcher parentWatcher : parentWatchers ) { - isConnected.set(newIsConnected); - connectionStartMs = System.currentTimeMillis(); - if ( newIsConnected ) - { - lastNegotiatedSessionTimeoutMs.set(zooKeeper.getNegotiatedSessionTimeoutMs()); - log.debug("Negotiated session timeout: " + lastNegotiatedSessionTimeoutMs.get()); - } + TimeTrace timeTrace = new TimeTrace("connection-state-parent-process", tracer.get()); + parentWatcher.process(event); + timeTrace.commit(); } } http://git-wip-us.apache.org/repos/asf/curator/blob/1bbf301a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestBlockUntilConnected.java --
[08/45] curator git commit: minor reformat
minor reformat Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/b8d4c3d7 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/b8d4c3d7 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/b8d4c3d7 Branch: refs/heads/CURATOR-248 Commit: b8d4c3d77de029917820634fa4ed21be19bbcf2c Parents: d317009 Author: randgalt Authored: Fri Aug 21 17:59:07 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:59:07 2015 -0500 -- .../src/test/java/org/apache/curator/TestEnsurePath.java | 2 ++ 1 file changed, 2 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/b8d4c3d7/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java -- diff --git a/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java b/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java index 4fe9561..871e4af 100644 --- a/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java +++ b/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java @@ -51,6 +51,7 @@ public class TestEnsurePath CuratorZookeeperClient curator = mock(CuratorZookeeperClient.class); RetryPolicy retryPolicy = new RetryOneTime(1); RetryLoop retryLoop = new RetryLoop(retryPolicy, null); +when(curator.retryConnectionTimeouts()).thenReturn(true); when(curator.getZooKeeper()).thenReturn(client); when(curator.getRetryPolicy()).thenReturn(retryPolicy); when(curator.newRetryLoop()).thenReturn(retryLoop); @@ -76,6 +77,7 @@ public class TestEnsurePath RetryPolicy retryPolicy = new RetryOneTime(1); RetryLoop retryLoop = new RetryLoop(retryPolicy, null); final CuratorZookeeperClient curator = mock(CuratorZookeeperClient.class); +when(curator.retryConnectionTimeouts()).thenReturn(true); when(curator.getZooKeeper()).thenReturn(client); when(curator.getRetryPolicy()).thenReturn(retryPolicy); when(curator.newRetryLoop()).thenReturn(retryLoop);
[39/45] curator git commit: Merge branch 'master' into CURATOR-3.0
Merge branch 'master' into CURATOR-3.0 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/d01eabe5 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/d01eabe5 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/d01eabe5 Branch: refs/heads/CURATOR-248 Commit: d01eabe5b3ff7624fa2bd423179316931d94c1e5 Parents: fed0bee 3801167 Author: randgalt Authored: Mon Aug 31 07:19:53 2015 -0700 Committer: randgalt Committed: Mon Aug 31 07:19:53 2015 -0700 -- .../org/apache/curator/retry/RetryForever.java | 60 .../java/org/apache/curator/TestRetryLoop.java | 20 +++ .../framework/recipes/cache/NodeCache.java | 41 ++--- curator-x-discovery-server/pom.xml | 6 ++ curator-x-discovery/pom.xml | 6 ++ curator-x-rpc/pom.xml | 6 ++ 6 files changed, 132 insertions(+), 7 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/d01eabe5/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java -- diff --cc curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java index 49b9a3f,bfc27d8..8b70db1 --- a/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java +++ b/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java @@@ -22,11 -22,10 +22,11 @@@ import com.google.common.annotations.Vi import com.google.common.base.Function; import com.google.common.base.Objects; import com.google.common.base.Preconditions; + import org.apache.curator.framework.CuratorFramework; +import org.apache.curator.framework.WatcherRemoveCuratorFramework; import org.apache.curator.framework.api.BackgroundCallback; import org.apache.curator.framework.api.CuratorEvent; - import org.apache.curator.framework.api.CuratorWatcher; import org.apache.curator.framework.listen.ListenerContainer; import org.apache.curator.framework.state.ConnectionState; import org.apache.curator.framework.state.ConnectionStateListener; @@@ -170,10 -178,17 +179,18 @@@ public class NodeCache implements Close { if ( state.compareAndSet(State.STARTED, State.CLOSED) ) { +client.removeWatchers(); listeners.clear(); - } - client.getConnectionStateListenable().removeListener(connectionStateListener); + client.clearWatcherReferences(watcher); + client.getConnectionStateListenable().removeListener(connectionStateListener); + + // TODO + // From PathChildrenCache + // This seems to enable even more GC - I'm not sure why yet - it + // has something to do with Guava's cache and circular references + connectionStateListener = null; + watcher = null; + } } /** http://git-wip-us.apache.org/repos/asf/curator/blob/d01eabe5/curator-x-discovery-server/pom.xml -- http://git-wip-us.apache.org/repos/asf/curator/blob/d01eabe5/curator-x-discovery/pom.xml -- http://git-wip-us.apache.org/repos/asf/curator/blob/d01eabe5/curator-x-rpc/pom.xml --
[44/45] curator git commit: Merge branch 'CURATOR-247' into CURATOR-3.0
Merge branch 'CURATOR-247' into CURATOR-3.0 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/d57aaeb7 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/d57aaeb7 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/d57aaeb7 Branch: refs/heads/CURATOR-248 Commit: d57aaeb77b40fd6a4f5aa52e3d4e44dbe02c1700 Parents: 667c97e c641c02 Author: randgalt Authored: Tue Sep 1 06:02:08 2015 -0700 Committer: randgalt Committed: Tue Sep 1 06:02:08 2015 -0700 -- .../org/apache/curator/ConnectionState.java | 108 +++ .../apache/curator/CuratorZookeeperClient.java | 112 +--- .../java/org/apache/curator/HandleHolder.java | 19 ++ .../main/java/org/apache/curator/RetryLoop.java | 18 +- .../ClassicConnectionHandlingPolicy.java| 68 +++ .../connection/ConnectionHandlingPolicy.java| 72 .../StandardConnectionHandlingPolicy.java | 56 ++ .../org/apache/curator/utils/DebugUtils.java| 11 +- .../java/org/apache/curator/TestEnsurePath.java | 3 + .../framework/CuratorFrameworkFactory.java | 58 ++ .../framework/api/UnhandledErrorListener.java | 4 +- .../imps/ClassicInternalConnectionHandler.java | 58 ++ .../framework/imps/CuratorFrameworkImpl.java| 86 - .../imps/InternalConnectionHandler.java | 10 ++ .../imps/StandardInternalConnectionHandler.java | 22 +++ .../framework/state/ConnectionState.java| 27 ++- .../framework/state/ConnectionStateManager.java | 91 +++--- .../framework/imps/TestBlockUntilConnected.java | 1 + .../imps/TestEnabledSessionExpiredState.java| 179 +++ .../framework/imps/TestFrameworkEdges.java | 2 - ...estResetConnectionWithBackgroundFailure.java | 19 +- .../recipes/leader/TestLeaderSelectorEdges.java | 6 +- .../locks/TestInterProcessMutexBase.java| 19 +- .../apache/curator/test/BaseClassForTests.java | 37 +++- .../java/org/apache/curator/test/Timing.java| 35 +++- .../discovery/details/TestServiceDiscovery.java | 2 + src/site/confluence/errors.confluence | 6 +- 27 files changed, 945 insertions(+), 184 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/d57aaeb7/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java --
[45/45] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-248
Merge branch 'CURATOR-3.0' into CURATOR-248 Conflicts: curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/c117b085 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/c117b085 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/c117b085 Branch: refs/heads/CURATOR-248 Commit: c117b0853b5779829ff732514aedd7d60b696ccc Parents: d412f23 d57aaeb Author: randgalt Authored: Tue Sep 1 06:27:31 2015 -0700 Committer: randgalt Committed: Tue Sep 1 06:27:31 2015 -0700 -- .../org/apache/curator/ConnectionState.java | 108 --- .../apache/curator/CuratorZookeeperClient.java | 112 +-- .../java/org/apache/curator/HandleHolder.java | 19 ++ .../main/java/org/apache/curator/RetryLoop.java | 18 +- .../ClassicConnectionHandlingPolicy.java| 68 + .../connection/ConnectionHandlingPolicy.java| 72 + .../StandardConnectionHandlingPolicy.java | 56 .../org/apache/curator/retry/RetryForever.java | 60 .../org/apache/curator/utils/DebugUtils.java| 11 +- .../java/org/apache/curator/TestEnsurePath.java | 3 + .../java/org/apache/curator/TestRetryLoop.java | 20 ++ .../framework/CuratorFrameworkFactory.java | 58 ...reateModeStatBackgroundPathAndBytesable.java | 25 ++ .../api/BackgroundPathableQuietlyable.java | 18 ++ .../api/CreateBackgroundModeStatACLable.java| 70 + .../curator/framework/api/CreateBuilder.java| 9 +- ...ateProtectACLCreateModePathAndBytesable.java | 72 + ...rotectACLCreateModeStatPathAndBytesable.java | 25 ++ .../framework/api/UnhandledErrorListener.java | 4 +- .../imps/ClassicInternalConnectionHandler.java | 58 .../framework/imps/CreateBuilderImpl.java | 299 ++- .../framework/imps/CuratorFrameworkImpl.java| 86 +++--- .../imps/InternalConnectionHandler.java | 10 + .../imps/StandardInternalConnectionHandler.java | 22 ++ .../framework/state/ConnectionState.java| 27 +- .../framework/state/ConnectionStateManager.java | 91 -- .../framework/imps/TestBlockUntilConnected.java | 1 + .../framework/imps/TestCreateReturningStat.java | 199 .../imps/TestEnabledSessionExpiredState.java| 179 +++ .../framework/imps/TestFrameworkEdges.java | 6 +- .../framework/recipes/cache/NodeCache.java | 41 ++- ...estResetConnectionWithBackgroundFailure.java | 19 +- .../recipes/leader/TestLeaderLatch.java | 15 +- .../recipes/leader/TestLeaderSelector.java | 5 +- .../recipes/leader/TestLeaderSelectorEdges.java | 6 +- .../locks/TestInterProcessMutexBase.java| 19 +- .../apache/curator/test/BaseClassForTests.java | 37 ++- .../java/org/apache/curator/test/Timing.java| 35 ++- curator-x-discovery-server/pom.xml | 6 + curator-x-discovery/pom.xml | 6 + .../discovery/details/TestServiceDiscovery.java | 2 + curator-x-rpc/pom.xml | 6 + src/site/confluence/errors.confluence | 6 +- src/site/confluence/index.confluence| 7 + 44 files changed, 1798 insertions(+), 218 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/c117b085/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --cc curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index 9a67684,daffa13..2b01b30 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@@ -31,8 -34,7 +34,9 @@@ import org.apache.curator.framework.imp import org.apache.curator.framework.imps.CuratorTempFrameworkImpl; import org.apache.curator.framework.imps.DefaultACLProvider; import org.apache.curator.framework.imps.GzipCompressionProvider; +import org.apache.curator.framework.state.ErrorPolicy; +import org.apache.curator.framework.state.StandardErrorPolicy; + import org.apache.curator.framework.state.ConnectionState; import org.apache.curator.utils.DefaultZookeeperFactory; import org.apache.curator.utils.ZookeeperFactory; import org.apache.zookeeper.CreateMode; @@@ -118,7 -121,7 +123,8 @@@ public class CuratorFrameworkFactor private ACLProvider aclProvider = DEFAULT_ACL_PROVIDER; private boolean canBeReadOnly = false; private boolean useContainerParentsIfAvailable = true; +private ErrorPolicy er
[11/45] curator git commit: removed some test code
removed some test code Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/6381ccb6 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/6381ccb6 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/6381ccb6 Branch: refs/heads/CURATOR-248 Commit: 6381ccb6536f4710248a50ae5d0313399bbfe858 Parents: ec2f9bd Author: randgalt Authored: Sat Aug 22 17:50:09 2015 -0500 Committer: randgalt Committed: Sat Aug 22 17:50:09 2015 -0500 -- .../src/main/java/org/apache/curator/RetryLoop.java | 10 +++--- .../main/java/org/apache/curator/utils/DebugUtils.java| 1 - .../java/org/apache/curator/test/BaseClassForTests.java | 5 - 3 files changed, 3 insertions(+), 13 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/6381ccb6/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index f6abf21..92291c1 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -74,7 +74,6 @@ public class RetryLoop unit.sleep(time); } }; -private static final boolean checkInjectedDebugExceptions = Boolean.getBoolean(DebugUtils.PROPERTY_CHECK_INJECTED_DEBUG_EXCEPTIONS); /** * Returns the default retry sleeper @@ -104,13 +103,10 @@ public class RetryLoop { try { -if ( checkInjectedDebugExceptions ) +Exception debugException = client.getDebugException(); +if ( debugException != null ) { -Exception debugException = client.getDebugException(); -if ( debugException != null ) -{ -throw debugException; -} +throw debugException; } client.internalBlockUntilConnectedOrTimedOut(); http://git-wip-us.apache.org/repos/asf/curator/blob/6381ccb6/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java -- diff --git a/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java b/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java index 0e473fb..3e90600 100644 --- a/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java +++ b/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java @@ -26,7 +26,6 @@ public class DebugUtils public static final String PROPERTY_LOG_ONLY_FIRST_CONNECTION_ISSUE_AS_ERROR_LEVEL = "curator-log-only-first-connection-issue-as-error-level"; public static final String PROPERTY_REMOVE_WATCHERS_IN_FOREGROUND = "curator-remove-watchers-in-foreground"; public static final String PROPERTY_RETRY_FAILED_TESTS = "curator-retry-failed-tests"; -public static final String PROPERTY_CHECK_INJECTED_DEBUG_EXCEPTIONS = "curator-check-injected-debug-exceptions"; private DebugUtils() { http://git-wip-us.apache.org/repos/asf/curator/blob/6381ccb6/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java -- diff --git a/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java b/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java index 55dcb61..c9f3524 100644 --- a/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java +++ b/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java @@ -44,20 +44,17 @@ public class BaseClassForTests private static final String INTERNAL_PROPERTY_DONT_LOG_CONNECTION_ISSUES; private static final String INTERNAL_PROPERTY_REMOVE_WATCHERS_IN_FOREGROUND; private static final String INTERNAL_RETRY_FAILED_TESTS; -private static final String INTERNAL_CHECK_INJECTED_DEBUG_EXCEPTIONS; static { String logConnectionIssues = null; String retryFailedTests = null; -String checkInjectedDebugExceptions = null; try { // use reflection to avoid adding a circular dependency in the pom Class debugUtilsClazz = Class.forName("org.apache.curator.utils.DebugUtils"); logConnectionIssues = (String)debugUtilsClazz.getField("PROPERTY_DONT_LOG_CONNECTION_ISSUES").get(null); retryFailedTests = (String)debugUtilsClazz.getField("PROPERTY_RETRY_FAILED_TESTS").get(null); -checkInjectedDebugExceptions = (String)debugUtilsClazz.getFie
[07/45] curator git commit: more tests
more tests Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/d3170099 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/d3170099 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/d3170099 Branch: refs/heads/CURATOR-248 Commit: d3170099757c7e17ff8fbee0c37d620aacb60d65 Parents: c5a4921 Author: randgalt Authored: Fri Aug 21 17:49:55 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:49:55 2015 -0500 -- .../imps/TestEnabledSessionExpiredState.java| 24 1 file changed, 24 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/d3170099/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java index 4c5ff81..150eb50 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java @@ -109,6 +109,30 @@ public class TestEnabledSessionExpiredState extends BaseClassForTests Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.RECONNECTED); } +@Test +public void testSessionExpirationFromTimeout() throws Exception +{ +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED); +server.stop(); +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.SUSPENDED); +Assert.assertEquals(states.poll(timing.multiple(2).session(), TimeUnit.MILLISECONDS), ConnectionState.LOST); +} + +@Test +public void testSessionExpirationFromTimeoutWithRestart() throws Exception +{ +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED); +server.stop(); +Thread.sleep(timing.multiple(1.2).session()); +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.SUSPENDED); +Assert.assertEquals(states.poll(timing.multiple(2).session(), TimeUnit.MILLISECONDS), ConnectionState.LOST); +server.restart(); +client.checkExists().forPath("/"); +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.RECONNECTED); + +Assert.assertNull(states.poll(timing.multiple(.5).milliseconds(), TimeUnit.MILLISECONDS)); // there should be no other events +} + @Override protected boolean enabledSessionExpiredStateAware() {
[21/45] curator git commit: more work on repairing tests for new connection handling
more work on repairing tests for new connection handling Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/e001e009 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/e001e009 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/e001e009 Branch: refs/heads/CURATOR-248 Commit: e001e0098f64baa8e0b3b887507bc98972c775dc Parents: 5f094f8 Author: randgalt Authored: Sun Aug 23 09:33:46 2015 -0500 Committer: randgalt Committed: Sun Aug 23 09:33:46 2015 -0500 -- .../org/apache/curator/CuratorZookeeperClient.java| 14 -- .../src/main/java/org/apache/curator/RetryLoop.java | 6 -- .../curator/framework/imps/TestFrameworkEdges.java| 3 --- .../x/discovery/details/TestServiceDiscovery.java | 2 ++ 4 files changed, 2 insertions(+), 23 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/e001e009/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index c8a9936..d6c2072 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -19,7 +19,6 @@ package org.apache.curator; -import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Preconditions; import org.apache.curator.connection.ClassicConnectionHandlingPolicy; import org.apache.curator.connection.ConnectionHandlingPolicy; @@ -54,7 +53,6 @@ public class CuratorZookeeperClient implements Closeable private final AtomicBoolean started = new AtomicBoolean(false); private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); private final ConnectionHandlingPolicy connectionHandlingPolicy; -private final AtomicReference debugException = new AtomicReference<>(); /** * @@ -336,18 +334,6 @@ public class CuratorZookeeperClient implements Closeable return connectionHandlingPolicy; } -@VisibleForTesting -public void setDebugException(Exception e) -{ -debugException.set(e); -} - -@VisibleForTesting -Exception getDebugException() -{ -return debugException.get(); -} - void addParentWatcher(Watcher watcher) { state.addParentWatcher(watcher); http://git-wip-us.apache.org/repos/asf/curator/blob/e001e009/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index 4353c61..e58ce20 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -96,12 +96,6 @@ public class RetryLoop */ public static T callWithRetry(CuratorZookeeperClient client, Callable proc) throws Exception { -Exception debugException = client.getDebugException(); -if ( debugException != null ) -{ -throw debugException; -} - return client.getConnectionHandlingPolicy().callWithRetry(client, proc); } http://git-wip-us.apache.org/repos/asf/curator/blob/e001e009/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java index 15781dc..ef77087 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java @@ -423,7 +423,6 @@ public class TestFrameworkEdges extends BaseClassForTests { throw new Error(e); } -client.getZookeeperClient().setDebugException(null); } try { @@ -441,7 +440,6 @@ public class TestFrameworkEdges extends BaseClassForTests server.stop(); // test foreground retry -client.getZookeeperClient().setDebugException(new KeeperException.ConnectionLossException()); client.checkExists().forPath("/hey"); Assert.assertTrue(semaphore.tryAcquire(MAX_RETRIES,
[03/45] curator git commit: First pass at new (optional) definition of state LOST
First pass at new (optional) definition of state LOST Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/344634ac Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/344634ac Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/344634ac Branch: refs/heads/CURATOR-248 Commit: 344634ac6e34e61bc0cc7b41923a1df4089c7948 Parents: 7d97259 Author: randgalt Authored: Fri Aug 21 12:10:24 2015 -0500 Committer: randgalt Committed: Fri Aug 21 12:10:24 2015 -0500 -- .../framework/CuratorFrameworkFactory.java | 19 + .../framework/api/UnhandledErrorListener.java | 4 +- .../framework/imps/CuratorFrameworkImpl.java| 10 ++- .../framework/state/ConnectionState.java| 35 +++-- .../framework/state/ConnectionStateManager.java | 75 +++- 5 files changed, 113 insertions(+), 30 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/344634ac/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index dcb2ee6..6209b06 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@ -31,6 +31,7 @@ import org.apache.curator.framework.imps.CuratorFrameworkImpl; import org.apache.curator.framework.imps.CuratorTempFrameworkImpl; import org.apache.curator.framework.imps.DefaultACLProvider; import org.apache.curator.framework.imps.GzipCompressionProvider; +import org.apache.curator.framework.state.ConnectionState; import org.apache.curator.utils.DefaultZookeeperFactory; import org.apache.curator.utils.ZookeeperFactory; import org.apache.zookeeper.CreateMode; @@ -116,6 +117,7 @@ public class CuratorFrameworkFactory private ACLProvider aclProvider = DEFAULT_ACL_PROVIDER; private boolean canBeReadOnly = false; private boolean useContainerParentsIfAvailable = true; +private boolean enableSessionExpiredState = false; /** * Apply the current values and build a new CuratorFramework @@ -343,6 +345,18 @@ public class CuratorFrameworkFactory return this; } +/** + * Changes the meaning of {@link ConnectionState#LOST} from it's pre Curator 3.0.0 meaning + * to a true lost session state. See the {@link ConnectionState#LOST} doc for details. + * + * @return this + */ +public Builder enableSessionExpiredState() +{ +this.enableSessionExpiredState = true; +return this; +} + public ACLProvider getAclProvider() { return aclProvider; @@ -398,6 +412,11 @@ public class CuratorFrameworkFactory return useContainerParentsIfAvailable; } +public boolean getEnableSessionExpiredState() +{ +return enableSessionExpiredState; +} + @Deprecated public String getAuthScheme() { http://git-wip-us.apache.org/repos/asf/curator/blob/344634ac/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java b/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java index b463af2..3721d4b 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java @@ -24,9 +24,7 @@ import org.apache.curator.framework.state.ConnectionStateListener; public interface UnhandledErrorListener { /** - * Called when an exception is caught in a background thread, handler, etc. Before this - * listener is called, the error will have been logged and a {@link ConnectionState#LOST} - * event will have been queued for any {@link ConnectionStateListener}s. + * Called when an exception is caught in a background thread, handler, etc. * * @param message Source message * @param e exception http://git-wip-us.apache.org/repos/asf/curator/blob/344634ac/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/fr
[32/45] curator git commit: minor typo
minor typo Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/bc57786c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/bc57786c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/bc57786c Branch: refs/heads/CURATOR-248 Commit: bc57786c9c55f06f8918e0e2255277bfb2fbbecb Parents: b8240a1 Author: randgalt Authored: Mon Aug 24 11:11:16 2015 -0500 Committer: randgalt Committed: Mon Aug 24 11:11:16 2015 -0500 -- src/site/confluence/errors.confluence | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/bc57786c/src/site/confluence/errors.confluence -- diff --git a/src/site/confluence/errors.confluence b/src/site/confluence/errors.confluence index 0111cf4..a744128 100644 --- a/src/site/confluence/errors.confluence +++ b/src/site/confluence/errors.confluence @@ -26,7 +26,7 @@ appropriate action. These are the possible state changes: |READ_ONLY|The connection has gone into read\-only mode. This can only happen if you pass true for CuratorFrameworkFactory.Builder.canBeReadOnly(). See the ZooKeeper doc regarding read only connections: [[http://wiki.apache.org/hadoop/ZooKeeper/GSoCReadOnlyMode]]. The connection will remain in read only mode until another state change is sent.| |SUSPENDED|There has been a loss of connection. Leaders, locks, etc. should suspend until the connection is re\-established.| |RECONNECTED|A suspended or lost connection has been re\-established.| -|LOST|Curator will set the LOST state when it believes that the ZooKeeper session has expired. ZooKeeper connections have a session. When the session expires, clients must take appropriate action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper connection. Curator will set the LOST state when any of the following occurs: a) ZooKeeper returns a Watcher.Event.KeeperState.Expired or KeeperException.Code.SESSIONEXPIRED; b) Curator closes the internally managed ZooKeeper instance; c) The session timeout elapses during a network partition. It is possible to get a RECONNECTED state after this but you should still consider any locks, etc. as dirty/unstable. *NOTE* The meaning of LOST has changed since Curator 3.0.0. Prior to 3.0.0 LOST only meant that the retry policy had expired.| +|LOST|Curator will set the LOST state when it believes that the ZooKeeper session has expired. ZooKeeper connections have a session. When the session expires, clients must take appropriate action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper connection. Curator will set the LOST state when any of the following occurs: a) ZooKeeper returns a Watcher.Event.KeeperState.Expired or KeeperException.Code.SESSIONEXPIRED; b) Curator closes the internally managed ZooKeeper instance; c) The session timeout elapses during a network partition. It is possible to get a RECONNECTED state after this but you should still consider any locks, etc. as dirty/unstable. *NOTE*: The meaning of LOST has changed since Curator 3.0.0. Prior to 3.0.0 LOST only meant that the retry policy had expired.| {{UnhandledErrorListener}} is called when a background task, etc. catches an exception. In general, Curator users shouldn't care about these as they are logged. However, you can listen for them if you choose.
[26/45] curator git commit: code for useSessionTimeoutMs was wrong
code for useSessionTimeoutMs was wrong Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/dd788163 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/dd788163 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/dd788163 Branch: refs/heads/CURATOR-248 Commit: dd788163a278e2c10f71d2f0fd8b68137806f9f0 Parents: 966b8df Author: randgalt Authored: Sun Aug 23 20:33:55 2015 -0500 Committer: randgalt Committed: Sun Aug 23 20:33:55 2015 -0500 -- .../apache/curator/framework/state/ConnectionStateManager.java| 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/dd788163/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index 3da7534..de71264 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -292,7 +292,8 @@ public class ConnectionStateManager implements Closeable if ( (currentConnectionState == ConnectionState.SUSPENDED) && (startOfSuspendedEpoch != 0) ) { long elapsedMs = System.currentTimeMillis() - startOfSuspendedEpoch; -int useSessionTimeoutMs = Math.max(client.getZookeeperClient().getLastNegotiatedSessionTimeoutMs(), sessionTimeoutMs); +int lastNegotiatedSessionTimeoutMs = client.getZookeeperClient().getLastNegotiatedSessionTimeoutMs(); +int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; if ( elapsedMs >= useSessionTimeoutMs ) { log.warn(String.format("Session timeout has elapsed while SUSPENDED. Posting LOST event and resetting the connection. Elapsed ms: %d. Session Timeout ms: %d", elapsedMs, useSessionTimeoutMs));
[13/45] curator git commit: Merge remote-tracking branch 'origin/CURATOR-3.0' into CURATOR-214
Merge remote-tracking branch 'origin/CURATOR-3.0' into CURATOR-214 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/ba4b5d8c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/ba4b5d8c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/ba4b5d8c Branch: refs/heads/CURATOR-248 Commit: ba4b5d8cb1f9733d3901b0b619528454d3dbf8c8 Parents: af369de 2fc9e37 Author: Cam McKenzie Authored: Sun Aug 23 09:35:05 2015 +1000 Committer: Cam McKenzie Committed: Sun Aug 23 09:35:05 2015 +1000 -- .../recipes/queue/DistributedIdQueue.java | 28 +++- .../recipes/queue/DistributedQueue.java | 7 - .../recipes/queue/TestDistributedIdQueue.java | 2 +- .../recipes/queue/TestDistributedQueue.java | 2 +- 4 files changed, 35 insertions(+), 4 deletions(-) --
[35/45] curator git commit: Include a note about 2 versions of Curator
Include a note about 2 versions of Curator Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/c9cb199b Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/c9cb199b Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/c9cb199b Branch: refs/heads/CURATOR-248 Commit: c9cb199b07b2047487802d6d5eb90a9e96104008 Parents: 0f5668b Author: randgalt Authored: Sun Aug 30 14:04:12 2015 -0700 Committer: randgalt Committed: Sun Aug 30 14:04:12 2015 -0700 -- src/site/confluence/index.confluence | 7 +++ 1 file changed, 7 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/c9cb199b/src/site/confluence/index.confluence -- diff --git a/src/site/confluence/index.confluence b/src/site/confluence/index.confluence index cd533cf..0f0e2dc 100644 --- a/src/site/confluence/index.confluence +++ b/src/site/confluence/index.confluence @@ -33,3 +33,10 @@ most users, the only artifact you need is curator\-recipes. |org.apache.curator|curator\-x\-discovery|A Service Discovery implementation built on the Curator Framework.| |org.apache.curator|curator\-x\-discovery-server|A RESTful server that can be used with Curator Discovery.| |org.apache.curator|curator\-x\-rpc|A proxy that bridges non\-java environments with the Curator framework and recipes.| + +h2. Versions + +The are currently two released versions of Curator, 2.x.x and 3.x.x: + +* Curator 2.x.x \- compatible with both ZooKeeper 3.4.x and ZooKeeper 3.5.x +* Curator 3.x.x \- compatible only with ZooKeeper 3.5.x and includes support for new features such as containers, dynamic reconfiguration, etc.
[09/45] curator git commit: wip
wip Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/847cc0d2 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/847cc0d2 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/847cc0d2 Branch: refs/heads/CURATOR-248 Commit: 847cc0d2415f59c2943d4a2734564119ffb38bb1 Parents: b8d4c3d Author: randgalt Authored: Sat Aug 22 10:47:01 2015 -0500 Committer: randgalt Committed: Sat Aug 22 10:47:01 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 15 ++-- .../apache/curator/CuratorZookeeperClient.java | 36 ++-- .../framework/imps/CuratorFrameworkImpl.java| 12 ++- .../framework/state/ConnectionStateManager.java | 2 +- .../imps/TestEnabledSessionExpiredState.java| 2 +- ...estResetConnectionWithBackgroundFailure.java | 19 +++ .../java/org/apache/curator/test/Timing.java| 21 7 files changed, 84 insertions(+), 23 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/847cc0d2/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index 1dfdbef..c3d6921 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -52,6 +52,7 @@ class ConnectionState implements Watcher, Closeable private final Queue parentWatchers = new ConcurrentLinkedQueue(); private final AtomicLong instanceIndex = new AtomicLong(); private volatile long connectionStartMs = 0; +private final AtomicBoolean enableTimeoutChecks = new AtomicBoolean(true); ConnectionState(ZookeeperFactory zookeeperFactory, EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher parentWatcher, AtomicReference tracer, boolean canBeReadOnly) { @@ -67,6 +68,11 @@ class ConnectionState implements Watcher, Closeable zooKeeper = new HandleHolder(zookeeperFactory, this, ensembleProvider, sessionTimeoutMs, canBeReadOnly); } +void disableTimeoutChecks() +{ +enableTimeoutChecks.set(false); +} + ZooKeeper getZooKeeper() throws Exception { if ( SessionFailRetryLoop.sessionForThreadHasFailed() ) @@ -81,10 +87,13 @@ class ConnectionState implements Watcher, Closeable throw exception; } -boolean localIsConnected = isConnected.get(); -if ( !localIsConnected ) +if ( enableTimeoutChecks.get() ) { -checkTimeouts(); +boolean localIsConnected = isConnected.get(); +if ( !localIsConnected ) +{ +checkTimeouts(); +} } return zooKeeper.getZooKeeper(); http://git-wip-us.apache.org/repos/asf/curator/blob/847cc0d2/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index fbb2f4c..ce6e9d3 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -50,6 +50,7 @@ public class CuratorZookeeperClient implements Closeable private final int connectionTimeoutMs; private final AtomicBoolean started = new AtomicBoolean(false); private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); +private final boolean manageTimeouts; /** * @@ -61,7 +62,7 @@ public class CuratorZookeeperClient implements Closeable */ public CuratorZookeeperClient(String connectString, int sessionTimeoutMs, int connectionTimeoutMs, Watcher watcher, RetryPolicy retryPolicy) { -this(new DefaultZookeeperFactory(), new FixedEnsembleProvider(connectString), sessionTimeoutMs, connectionTimeoutMs, watcher, retryPolicy, false); +this(new DefaultZookeeperFactory(), new FixedEnsembleProvider(connectString), sessionTimeoutMs, connectionTimeoutMs, watcher, retryPolicy, false, true); } /** @@ -73,7 +74,7 @@ public class CuratorZookeeperClient implements Closeable */ public CuratorZookeeperClient(EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher watcher, RetryPolicy retryPolicy) { -this(new DefaultZookeeperFactory(), ensembleProvider, sessionTimeoutMs, connectionTimeoutMs, watcher, retryP
[04/45] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-247
Merge branch 'CURATOR-3.0' into CURATOR-247 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/2343daf2 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/2343daf2 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/2343daf2 Branch: refs/heads/CURATOR-248 Commit: 2343daf29388566b0efa0b0a2ad21574fb534a27 Parents: 344634a 2fc9e37 Author: randgalt Authored: Fri Aug 21 15:11:59 2015 -0500 Committer: randgalt Committed: Fri Aug 21 15:11:59 2015 -0500 -- .../recipes/queue/DistributedIdQueue.java | 28 +++- .../recipes/queue/DistributedQueue.java | 7 - .../recipes/queue/TestDistributedIdQueue.java | 2 +- .../recipes/queue/TestDistributedQueue.java | 2 +- 4 files changed, 35 insertions(+), 4 deletions(-) --
[40/45] curator git commit: Merge branch 'CURATOR-214' into CURATOR-3.0
Merge branch 'CURATOR-214' into CURATOR-3.0 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/667c97e7 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/667c97e7 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/667c97e7 Branch: refs/heads/CURATOR-248 Commit: 667c97e7f3e2966153fb5c105020deaac7d0f510 Parents: d01eabe 6c5a154 Author: Cam McKenzie Authored: Tue Sep 1 09:35:26 2015 +1000 Committer: Cam McKenzie Committed: Tue Sep 1 09:35:26 2015 +1000 -- ...reateModeStatBackgroundPathAndBytesable.java | 25 ++ .../api/CreateBackgroundModeStatACLable.java| 70 + .../curator/framework/api/CreateBuilder.java| 9 +- ...ateProtectACLCreateModePathAndBytesable.java | 72 + ...rotectACLCreateModeStatPathAndBytesable.java | 25 ++ .../framework/imps/CreateBuilderImpl.java | 299 ++- .../framework/imps/TestCreateReturningStat.java | 199 .../framework/imps/TestFrameworkEdges.java | 4 +- 8 files changed, 683 insertions(+), 20 deletions(-) --
[33/45] curator git commit: CURATOR-167 - Partial fix to clean up Curator managed watch objects when the cache closes. A full fix requires the ability to cancel watches in ZK which is not available un
CURATOR-167 - Partial fix to clean up Curator managed watch objects when the cache closes. A full fix requires the ability to cancel watches in ZK which is not available until ZK 3.5 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/8fae7856 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/8fae7856 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/8fae7856 Branch: refs/heads/CURATOR-248 Commit: 8fae7856edc1a5269fd012c683860e0b150e13b3 Parents: 5dc27c1 Author: Cameron McKenzie Authored: Wed Nov 19 14:23:24 2014 +1100 Committer: Cam McKenzie Committed: Tue Aug 25 14:06:35 2015 +1000 -- .../framework/recipes/cache/NodeCache.java | 41 1 file changed, 34 insertions(+), 7 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/8fae7856/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java -- diff --git a/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java b/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java index 72ee5ff..bfc27d8 100644 --- a/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java +++ b/curator-recipes/src/main/java/org/apache/curator/framework/recipes/cache/NodeCache.java @@ -22,19 +22,21 @@ import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Function; import com.google.common.base.Objects; import com.google.common.base.Preconditions; + import org.apache.curator.framework.CuratorFramework; import org.apache.curator.framework.api.BackgroundCallback; import org.apache.curator.framework.api.CuratorEvent; -import org.apache.curator.framework.api.CuratorWatcher; import org.apache.curator.framework.listen.ListenerContainer; import org.apache.curator.framework.state.ConnectionState; import org.apache.curator.framework.state.ConnectionStateListener; import org.apache.curator.utils.PathUtils; import org.apache.zookeeper.KeeperException; import org.apache.zookeeper.WatchedEvent; +import org.apache.zookeeper.Watcher; import org.apache.zookeeper.data.Stat; import org.slf4j.Logger; import org.slf4j.LoggerFactory; + import java.io.Closeable; import java.io.IOException; import java.util.concurrent.Exchanger; @@ -60,7 +62,7 @@ public class NodeCache implements Closeable private final AtomicReference state = new AtomicReference(State.LATENT); private final ListenerContainer listeners = new ListenerContainer(); private final AtomicBoolean isConnected = new AtomicBoolean(true); -private final ConnectionStateListener connectionStateListener = new ConnectionStateListener() +private ConnectionStateListener connectionStateListener = new ConnectionStateListener() { @Override public void stateChanged(CuratorFramework client, ConnectionState newState) @@ -86,12 +88,19 @@ public class NodeCache implements Closeable } }; -private final CuratorWatcher watcher = new CuratorWatcher() +private Watcher watcher = new Watcher() { @Override -public void process(WatchedEvent event) throws Exception +public void process(WatchedEvent event) { -reset(); +try +{ +reset(); +} +catch(Exception e) +{ +handleException(e); +} } }; @@ -170,8 +179,16 @@ public class NodeCache implements Closeable if ( state.compareAndSet(State.STARTED, State.CLOSED) ) { listeners.clear(); -} - client.getConnectionStateListenable().removeListener(connectionStateListener); +client.clearWatcherReferences(watcher); + client.getConnectionStateListenable().removeListener(connectionStateListener); + +// TODO +// From PathChildrenCache +// This seems to enable even more GC - I'm not sure why yet - it +// has something to do with Guava's cache and circular references +connectionStateListener = null; +watcher = null; +} } /** @@ -312,4 +329,14 @@ public class NodeCache implements Closeable } } } + +/** + * Default behavior is just to log the exception + * + * @param e the exception + */ +protected void handleException(Throwable e) +{ +log.error("", e); +} }
[18/45] curator git commit: Longer connection timeout needed
Longer connection timeout needed Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/face4034 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/face4034 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/face4034 Branch: refs/heads/CURATOR-248 Commit: face4034e9fdcc9ffdb394c7c1682add834a1e10 Parents: 05d241d Author: randgalt Authored: Sat Aug 22 21:54:24 2015 -0500 Committer: randgalt Committed: Sat Aug 22 21:54:24 2015 -0500 -- .../framework/recipes/leader/TestLeaderSelectorEdges.java| 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/face4034/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java -- diff --git a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java index d13fea7..03708f1 100644 --- a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java +++ b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java @@ -159,7 +159,7 @@ public class TestLeaderSelectorEdges extends BaseClassForTests CuratorFrameworkFactory.builder() .connectString(server.getConnectString()) .retryPolicy(new RetryNTimes(2, 1)) -.connectionTimeoutMs(100) +.connectionTimeoutMs(1000) .sessionTimeoutMs(6) .build(); final CountDownLatch latch = new CountDownLatch(1); @@ -211,7 +211,7 @@ public class TestLeaderSelectorEdges extends BaseClassForTests CuratorFrameworkFactory.builder() .connectString(server.getConnectString()) .retryPolicy(new RetryNTimes(0, 0)) -.connectionTimeoutMs(100) +.connectionTimeoutMs(1000) .sessionTimeoutMs(6) .build(); final CountDownLatch latch = new CountDownLatch(1);
[10/45] curator git commit: Fixed testRetry() for new LOST behavior
Fixed testRetry() for new LOST behavior Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/ec2f9bd5 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/ec2f9bd5 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/ec2f9bd5 Branch: refs/heads/CURATOR-248 Commit: ec2f9bd555d01b324bd5ef690b1036d98e1f3702 Parents: 847cc0d Author: randgalt Authored: Sat Aug 22 11:06:33 2015 -0500 Committer: randgalt Committed: Sat Aug 22 11:06:33 2015 -0500 -- .../org/apache/curator/CuratorZookeeperClient.java | 17 +++-- .../main/java/org/apache/curator/RetryLoop.java| 12 +++- .../java/org/apache/curator/utils/DebugUtils.java | 12 +++- .../curator/framework/imps/TestFrameworkEdges.java | 4 +++- .../org/apache/curator/test/BaseClassForTests.java | 5 + 5 files changed, 41 insertions(+), 9 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/ec2f9bd5/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index ce6e9d3..a065d78 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -19,6 +19,7 @@ package org.apache.curator; +import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Preconditions; import org.apache.curator.drivers.TracerDriver; import org.apache.curator.ensemble.EnsembleProvider; @@ -51,6 +52,7 @@ public class CuratorZookeeperClient implements Closeable private final AtomicBoolean started = new AtomicBoolean(false); private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); private final boolean manageTimeouts; +private final AtomicReference debugException = new AtomicReference<>(); /** * @@ -207,8 +209,7 @@ public class CuratorZookeeperClient implements Closeable if ( !started.compareAndSet(false, true) ) { -IllegalStateException ise = new IllegalStateException("Already started"); -throw ise; +throw new IllegalStateException("Already started"); } state.start(); @@ -337,6 +338,18 @@ public class CuratorZookeeperClient implements Closeable return manageTimeouts; } +@VisibleForTesting +public void setDebugException(Exception e) +{ +debugException.set(e); +} + +@VisibleForTesting +Exception getDebugException() +{ +return debugException.get(); +} + void addParentWatcher(Watcher watcher) { state.addParentWatcher(watcher); http://git-wip-us.apache.org/repos/asf/curator/blob/ec2f9bd5/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index 8d77cf7..f6abf21 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -74,6 +74,7 @@ public class RetryLoop unit.sleep(time); } }; +private static final boolean checkInjectedDebugExceptions = Boolean.getBoolean(DebugUtils.PROPERTY_CHECK_INJECTED_DEBUG_EXCEPTIONS); /** * Returns the default retry sleeper @@ -103,13 +104,22 @@ public class RetryLoop { try { +if ( checkInjectedDebugExceptions ) +{ +Exception debugException = client.getDebugException(); +if ( debugException != null ) +{ +throw debugException; +} +} + client.internalBlockUntilConnectedOrTimedOut(); if ( !client.isConnected() && !client.retryConnectionTimeouts() ) { connectionFailed = true; break; } - + result = proc.call(); retryLoop.markComplete(); } http://git-wip-us.apache.org/repos/asf/curator/blob/ec2f9bd5/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java -- diff --git a/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java b/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java
[30/45] curator git commit: CURATOR-234 Add remaining slf4j-log4j12 test deps
CURATOR-234 Add remaining slf4j-log4j12 test deps Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/b177ed70 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/b177ed70 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/b177ed70 Branch: refs/heads/CURATOR-248 Commit: b177ed709a422f0b06f3a4a3683e5488d038fbb9 Parents: 9a444fa Author: Mike Drob Authored: Mon Aug 24 10:40:12 2015 -0500 Committer: Mike Drob Committed: Mon Aug 24 10:41:09 2015 -0500 -- curator-x-discovery-server/pom.xml | 6 ++ curator-x-discovery/pom.xml| 6 ++ curator-x-rpc/pom.xml | 6 ++ 3 files changed, 18 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/b177ed70/curator-x-discovery-server/pom.xml -- diff --git a/curator-x-discovery-server/pom.xml b/curator-x-discovery-server/pom.xml index 2728ce5..4ed7e25 100644 --- a/curator-x-discovery-server/pom.xml +++ b/curator-x-discovery-server/pom.xml @@ -108,6 +108,12 @@ resteasy-jaxrs test + + +org.slf4j +slf4j-log4j12 +test + http://git-wip-us.apache.org/repos/asf/curator/blob/b177ed70/curator-x-discovery/pom.xml -- diff --git a/curator-x-discovery/pom.xml b/curator-x-discovery/pom.xml index 015fea7..9043533 100644 --- a/curator-x-discovery/pom.xml +++ b/curator-x-discovery/pom.xml @@ -66,5 +66,11 @@ testng test + + +org.slf4j +slf4j-log4j12 +test + http://git-wip-us.apache.org/repos/asf/curator/blob/b177ed70/curator-x-rpc/pom.xml -- diff --git a/curator-x-rpc/pom.xml b/curator-x-rpc/pom.xml index 299ef9f..519e8fb 100644 --- a/curator-x-rpc/pom.xml +++ b/curator-x-rpc/pom.xml @@ -118,6 +118,12 @@ testng test + + +org.slf4j +slf4j-log4j12 +test +
[16/45] curator git commit: Curator has a big problem with thread interrupted states getting cleared. There are several issues on this (CURATOR-208, CURATOR-205, CURATOR-228, CURATOR-109
Curator has a big problem with thread interrupted states getting cleared. There are several issues on this (CURATOR-208, CURATOR-205, CURATOR-228, CURATOR-109 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/23554479 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/23554479 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/23554479 Branch: refs/heads/CURATOR-248 Commit: 23554479597d654fa8318cdc579fc3cc29bc2c54 Parents: 30bd7b6 Author: randgalt Authored: Sat Aug 22 20:10:34 2015 -0500 Committer: randgalt Committed: Sat Aug 22 20:10:34 2015 -0500 -- .../org/apache/curator/framework/state/ConnectionStateManager.java | 2 +- .../java/org/apache/curator/framework/imps/TestFrameworkEdges.java | 1 - 2 files changed, 1 insertion(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/23554479/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index 406099d..44d511b 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -248,7 +248,7 @@ public class ConnectionStateManager implements Closeable { try { -while ( !Thread.currentThread().isInterrupted() ) +while ( !Thread.currentThread().isInterrupted() && (state.get() == State.STARTED) ) { int pollMaxMs = (sessionTimeoutMs * 2) / 3; // 2/3 of session timeout final ConnectionState newState = eventQueue.poll(pollMaxMs, TimeUnit.MILLISECONDS); http://git-wip-us.apache.org/repos/asf/curator/blob/23554479/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java index 7407eab..15781dc 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java @@ -195,7 +195,6 @@ public class TestFrameworkEdges extends BaseClassForTests @Test public void testGetAclNoStat() throws Exception { - CuratorFramework client = CuratorFrameworkFactory.newClient(server.getConnectString(), timing.session(), timing.connection(), new RetryOneTime(1)); client.start(); try
[06/45] curator git commit: license
license Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/c5a49216 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/c5a49216 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/c5a49216 Branch: refs/heads/CURATOR-248 Commit: c5a49216cc78b05201a8ded357e50e0b6313 Parents: 62f3c33 Author: randgalt Authored: Fri Aug 21 17:37:15 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:37:15 2015 -0500 -- .../api/BackgroundPathableQuietlyable.java| 18 ++ .../imps/TestEnabledSessionExpiredState.java | 18 ++ 2 files changed, 36 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/c5a49216/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java index 8ed73fa..cd88739 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java @@ -1,3 +1,21 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ package org.apache.curator.framework.api; public interface BackgroundPathableQuietlyable extends BackgroundPathable, Quietly> http://git-wip-us.apache.org/repos/asf/curator/blob/c5a49216/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java index 030a292..4c5ff81 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java @@ -1,3 +1,21 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ package org.apache.curator.framework.imps; import com.google.common.collect.Queues;
[34/45] curator git commit: CURATOR-161 Added missing license header
CURATOR-161 Added missing license header Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/0f5668b3 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/0f5668b3 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/0f5668b3 Branch: refs/heads/CURATOR-248 Commit: 0f5668b3127f280e6dd1d6d3ec665420e2d46b2d Parents: f9af0ce Author: Mike Drob Authored: Tue Aug 25 15:14:01 2015 -0500 Committer: Mike Drob Committed: Tue Aug 25 15:14:01 2015 -0500 -- .../api/BackgroundPathableQuietlyable.java| 18 ++ 1 file changed, 18 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/0f5668b3/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java index 8ed73fa..cd88739 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java @@ -1,3 +1,21 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ package org.apache.curator.framework.api; public interface BackgroundPathableQuietlyable extends BackgroundPathable, Quietly>
[20/45] curator git commit: More refinement of classic/new connection handling. Reworked how the retry policy is invoked for each. New behavior is now confirmed to be: wait for connection timeout only
More refinement of classic/new connection handling. Reworked how the retry policy is invoked for each. New behavior is now confirmed to be: wait for connection timeout only once. Some tests will need work due to this Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/5f094f8b Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/5f094f8b Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/5f094f8b Branch: refs/heads/CURATOR-248 Commit: 5f094f8bb6dca3c056051cb8800b418839cca0e1 Parents: face403 Author: randgalt Authored: Sun Aug 23 07:49:17 2015 -0500 Committer: randgalt Committed: Sun Aug 23 07:49:17 2015 -0500 -- .../apache/curator/CuratorZookeeperClient.java | 7 ++- .../main/java/org/apache/curator/RetryLoop.java | 57 ++-- .../ClassicConnectionHandlingPolicy.java| 29 +++--- .../connection/ConnectionHandlingPolicy.java| 37 +++-- .../StandardConnectionHandlingPolicy.java | 39 -- .../imps/TestEnabledSessionExpiredState.java| 6 +-- ...estResetConnectionWithBackgroundFailure.java | 10 ++-- .../locks/TestInterProcessMutexBase.java| 19 +++ .../java/org/apache/curator/test/Timing.java| 36 - 9 files changed, 99 insertions(+), 141 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/5f094f8b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index 9342acf..c8a9936 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -358,7 +358,12 @@ public class CuratorZookeeperClient implements Closeable state.removeParentWatcher(watcher); } -void internalBlockUntilConnectedOrTimedOut() throws InterruptedException +/** + * For internal use only + * + * @throws InterruptedException interruptions + */ +public void internalBlockUntilConnectedOrTimedOut() throws InterruptedException { long waitTimeMs = connectionTimeoutMs; while ( !state.isConnected() && (waitTimeMs > 0) ) http://git-wip-us.apache.org/repos/asf/curator/blob/5f094f8b/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index a17cbf3..4353c61 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -96,62 +96,13 @@ public class RetryLoop */ public static T callWithRetry(CuratorZookeeperClient client, Callable proc) throws Exception { -T result = null; -RetryLoop retryLoop = client.newRetryLoop(); -boolean connectionFailed = false; -while ( retryLoop.shouldContinue() ) +Exception debugException = client.getDebugException(); +if ( debugException != null ) { -try -{ -Exception debugException = client.getDebugException(); -if ( debugException != null ) -{ -throw debugException; -} - -client.internalBlockUntilConnectedOrTimedOut(); - -switch ( client.getConnectionHandlingPolicy().preRetry(client) ) -{ -default: -case CALL_PROC: -{ -result = proc.call(); -retryLoop.markComplete(); -break; -} - -case WAIT_FOR_CONNECTION: -{ -break; // just loop -} - -case EXIT_RETRIES: -{ -retryLoop.markComplete(); -break; -} - -case CONNECTION_TIMEOUT: -{ -connectionFailed = true; -retryLoop.markComplete(); -break; -} -} -} -catch ( Exception e ) -{ -retryLoop.takeException(e); -} -} - -if ( connectionFailed ) -{ -throw new KeeperException.ConnectionLossEx
[31/45] curator git commit: Merge branch 'CURATOR-234'
Merge branch 'CURATOR-234' Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/5dc27c1f Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/5dc27c1f Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/5dc27c1f Branch: refs/heads/CURATOR-248 Commit: 5dc27c1fa9b8e3bc6ed3e1b198bb1eca6b378c9a Parents: 061ed0a b177ed7 Author: Mike Drob Authored: Mon Aug 24 10:41:39 2015 -0500 Committer: Mike Drob Committed: Mon Aug 24 10:41:39 2015 -0500 -- curator-x-discovery-server/pom.xml | 6 ++ curator-x-discovery/pom.xml| 6 ++ curator-x-rpc/pom.xml | 6 ++ 3 files changed, 18 insertions(+) --
[37/45] curator git commit: Merge branch 'CURATOR-239' of github.com:leandronunes85/curator into CURATOR-239
Merge branch 'CURATOR-239' of github.com:leandronunes85/curator into CURATOR-239 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/35b20630 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/35b20630 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/35b20630 Branch: refs/heads/CURATOR-248 Commit: 35b206307447d9dcc8465c941f644ee9697e4723 Parents: 8fae785 3e3a810 Author: randgalt Authored: Mon Aug 31 07:17:23 2015 -0700 Committer: randgalt Committed: Mon Aug 31 07:17:23 2015 -0700 -- .../org/apache/curator/retry/RetryForever.java | 59 .../java/org/apache/curator/TestRetryLoop.java | 20 +++ 2 files changed, 79 insertions(+) --
[43/45] curator git commit: Use negotiated session timeout in processEvents()
Use negotiated session timeout in processEvents() Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/c641c029 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/c641c029 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/c641c029 Branch: refs/heads/CURATOR-248 Commit: c641c0296275bb12a156c2a6a1655e0cc968fad4 Parents: 667b1d2 Author: randgalt Authored: Mon Aug 31 19:50:21 2015 -0700 Committer: randgalt Committed: Mon Aug 31 19:50:21 2015 -0700 -- .../apache/curator/framework/state/ConnectionStateManager.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/c641c029/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index daa33f6..3d44d45 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -250,7 +250,9 @@ public class ConnectionStateManager implements Closeable { while ( !Thread.currentThread().isInterrupted() && (state.get() == State.STARTED) ) { -int pollMaxMs = (sessionTimeoutMs * 2) / 3; // 2/3 of session timeout +int lastNegotiatedSessionTimeoutMs = client.getZookeeperClient().getLastNegotiatedSessionTimeoutMs(); +int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; +int pollMaxMs = (useSessionTimeoutMs * 2) / 3; // 2/3 of session timeout final ConnectionState newState = eventQueue.poll(pollMaxMs, TimeUnit.MILLISECONDS); if ( newState != null ) {
[02/45] curator git commit: CURATOR-239 - Interrupting current thread after catching InterruptedException
CURATOR-239 - Interrupting current thread after catching InterruptedException Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/3e3a8103 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/3e3a8103 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/3e3a8103 Branch: refs/heads/CURATOR-248 Commit: 3e3a810312f996874c327f42c24aa5b10a91161a Parents: 6e8c108 Author: Leandro Nunes Authored: Fri Jul 24 00:18:18 2015 +0100 Committer: Leandro Nunes Committed: Fri Jul 24 00:18:18 2015 +0100 -- .../src/main/java/org/apache/curator/retry/RetryForever.java| 1 + 1 file changed, 1 insertion(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/3e3a8103/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java -- diff --git a/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java b/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java index 27444b9..89f0276 100644 --- a/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java +++ b/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java @@ -52,6 +52,7 @@ public class RetryForever implements RetryPolicy catch (InterruptedException e) { log.warn("Error occurred while sleeping", e); +Thread.currentThread().interrupt(); } return true; }
[23/45] curator git commit: doc
doc Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/64d966c1 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/64d966c1 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/64d966c1 Branch: refs/heads/CURATOR-248 Commit: 64d966c18b9d18c40e13fda98e52d9253b281086 Parents: 1a2a94b Author: randgalt Authored: Sun Aug 23 10:57:48 2015 -0500 Committer: randgalt Committed: Sun Aug 23 10:57:48 2015 -0500 -- .../ClassicConnectionHandlingPolicy.java| 3 +++ .../connection/ConnectionHandlingPolicy.java| 20 ++-- .../StandardConnectionHandlingPolicy.java | 5 + 3 files changed, 22 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/64d966c1/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java index d0db0bb..1d53728 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java @@ -4,6 +4,9 @@ import org.apache.curator.CuratorZookeeperClient; import org.apache.curator.RetryLoop; import java.util.concurrent.Callable; +/** + * Emulates the pre 3.0.0 Curator connection handling + */ public class ClassicConnectionHandlingPolicy implements ConnectionHandlingPolicy { @Override http://git-wip-us.apache.org/repos/asf/curator/blob/64d966c1/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java index 51bdccc..6cea67d 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java @@ -1,8 +1,13 @@ package org.apache.curator.connection; import org.apache.curator.CuratorZookeeperClient; +import org.apache.curator.RetryLoop; import java.util.concurrent.Callable; +/** + * Abstracts connection handling so that Curator can emulate it's old, pre 3.0.0 + * handling and update to newer handling. + */ public interface ConnectionHandlingPolicy { /** @@ -12,6 +17,15 @@ public interface ConnectionHandlingPolicy */ boolean isEmulatingClassicHandling(); +/** + * Called by {@link RetryLoop#callWithRetry(CuratorZookeeperClient, Callable)} to do the work + * of retrying + * + * @param client client + * @param proc the procedure to retry + * @return result + * @throws Exception errors + */ T callWithRetry(CuratorZookeeperClient client, Callable proc) throws Exception; enum CheckTimeoutsResult @@ -55,10 +69,4 @@ public interface ConnectionHandlingPolicy * @throws Exception errors */ CheckTimeoutsResult checkTimeouts(Callable hasNewConnectionString, long connectionStartMs, int sessionTimeoutMs, int connectionTimeoutMs) throws Exception; - -/* -int getDefaultConnectionTimeoutMs(); - -int getDefaultSessionTimeoutMs(); -*/ } http://git-wip-us.apache.org/repos/asf/curator/blob/64d966c1/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java index b16cd53..6906284 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java @@ -6,6 +6,11 @@ import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.util.concurrent.Callable; +/** + * Curator's standard connection handling since 3.0.0 + * + * @since 3.0.0 + */ public class StandardConnectionHandlingPolicy implements ConnectionHandlingPolicy { private final Logger log = LoggerFactory.getLogger(getClass());
[25/45] curator git commit: The true session timeout is a negotiated value between client and server. The new session timeout handling should use the negotiated value if available
The true session timeout is a negotiated value between client and server. The new session timeout handling should use the negotiated value if available Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/966b8dfc Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/966b8dfc Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/966b8dfc Branch: refs/heads/CURATOR-248 Commit: 966b8dfc8897f766f70a2334c90fa0913e06f996 Parents: 9c7cf5d Author: randgalt Authored: Sun Aug 23 20:31:21 2015 -0500 Committer: randgalt Committed: Sun Aug 23 20:31:21 2015 -0500 -- .../java/org/apache/curator/ConnectionState.java | 12 .../apache/curator/CuratorZookeeperClient.java | 10 ++ .../java/org/apache/curator/HandleHolder.java| 19 +++ .../framework/state/ConnectionStateManager.java | 5 +++-- 4 files changed, 44 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/966b8dfc/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index d6ddd33..555a52d 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -36,6 +36,7 @@ import java.util.Queue; import java.util.concurrent.Callable; import java.util.concurrent.ConcurrentLinkedQueue; import java.util.concurrent.atomic.AtomicBoolean; +import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicLong; import java.util.concurrent.atomic.AtomicReference; @@ -46,6 +47,7 @@ class ConnectionState implements Watcher, Closeable private final Logger log = LoggerFactory.getLogger(getClass()); private final HandleHolder zooKeeper; private final AtomicBoolean isConnected = new AtomicBoolean(false); +private final AtomicInteger lastNegotiatedSessionTimeoutMs = new AtomicInteger(0); private final EnsembleProvider ensembleProvider; private final int sessionTimeoutMs; private final int connectionTimeoutMs; @@ -141,6 +143,11 @@ class ConnectionState implements Watcher, Closeable return instanceIndex.get(); } +int getLastNegotiatedSessionTimeoutMs() +{ +return lastNegotiatedSessionTimeoutMs.get(); +} + @Override public void process(WatchedEvent event) { @@ -167,6 +174,11 @@ class ConnectionState implements Watcher, Closeable { isConnected.set(newIsConnected); connectionStartMs = System.currentTimeMillis(); +if ( newIsConnected ) +{ + lastNegotiatedSessionTimeoutMs.set(zooKeeper.getNegotiatedSessionTimeoutMs()); +log.debug("Negotiated session timeout: " + lastNegotiatedSessionTimeoutMs.get()); +} } } http://git-wip-us.apache.org/repos/asf/curator/blob/966b8dfc/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index d6c2072..471adf0 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -334,6 +334,16 @@ public class CuratorZookeeperClient implements Closeable return connectionHandlingPolicy; } +/** + * Return the most recent value of {@link ZooKeeper#getSessionTimeout()} or 0 + * + * @return session timeout or 0 + */ +public int getLastNegotiatedSessionTimeoutMs() +{ +return state.getLastNegotiatedSessionTimeoutMs(); +} + void addParentWatcher(Watcher watcher) { state.addParentWatcher(watcher); http://git-wip-us.apache.org/repos/asf/curator/blob/966b8dfc/curator-client/src/main/java/org/apache/curator/HandleHolder.java -- diff --git a/curator-client/src/main/java/org/apache/curator/HandleHolder.java b/curator-client/src/main/java/org/apache/curator/HandleHolder.java index 1f7cd91..8652f0c 100644 --- a/curator-client/src/main/java/org/apache/curator/HandleHolder.java +++ b/curator-client/src/main/java/org/apache/curator/HandleHolder.java @@ -39,6 +39,8 @@ class HandleHolder ZooKeeper getZooKeeper() throws Exception; String getConnectionString(); + +int getNegotiatedSess
[36/45] curator git commit: Include a note about 2 versions of Curator
Include a note about 2 versions of Curator Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/fed0bee8 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/fed0bee8 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/fed0bee8 Branch: refs/heads/CURATOR-248 Commit: fed0bee8450993bfae05d8ce9f350c1f9675de62 Parents: c9cb199 Author: randgalt Authored: Sun Aug 30 14:07:27 2015 -0700 Committer: randgalt Committed: Sun Aug 30 14:07:27 2015 -0700 -- src/site/confluence/index.confluence | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/fed0bee8/src/site/confluence/index.confluence -- diff --git a/src/site/confluence/index.confluence b/src/site/confluence/index.confluence index 0f0e2dc..4b5683e 100644 --- a/src/site/confluence/index.confluence +++ b/src/site/confluence/index.confluence @@ -39,4 +39,4 @@ h2. Versions The are currently two released versions of Curator, 2.x.x and 3.x.x: * Curator 2.x.x \- compatible with both ZooKeeper 3.4.x and ZooKeeper 3.5.x -* Curator 3.x.x \- compatible only with ZooKeeper 3.5.x and includes support for new features such as containers, dynamic reconfiguration, etc. +* Curator 3.x.x \- compatible only with ZooKeeper 3.5.x and includes support for new features such as dynamic reconfiguration, etc.
[17/45] curator git commit: When the connection timeout elapses and there is more than one server in the connection string, reset the connection and try again
When the connection timeout elapses and there is more than one server in the connection string, reset the connection and try again Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/05d241da Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/05d241da Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/05d241da Branch: refs/heads/CURATOR-248 Commit: 05d241da642c6ba0d16b3ce97557128fad4dfe41 Parents: 2355447 Author: randgalt Authored: Sat Aug 22 20:32:41 2015 -0500 Committer: randgalt Committed: Sat Aug 22 20:32:41 2015 -0500 -- .../src/main/java/org/apache/curator/RetryLoop.java| 5 + .../curator/connection/ConnectionHandlingPolicy.java | 5 + .../connection/StandardConnectionHandlingPolicy.java | 13 - 3 files changed, 22 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/05d241da/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index 35d55a1..a17cbf3 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -121,6 +121,11 @@ public class RetryLoop break; } +case WAIT_FOR_CONNECTION: +{ +break; // just loop +} + case EXIT_RETRIES: { retryLoop.markComplete(); http://git-wip-us.apache.org/repos/asf/curator/blob/05d241da/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java index f3ecce6..7f19159 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java @@ -63,6 +63,11 @@ public interface ConnectionHandlingPolicy CALL_PROC, /** + * Wait again for connection success or timeout + */ +WAIT_FOR_CONNECTION, + +/** * Do not call the procedure and exit the retry loop */ EXIT_RETRIES, http://git-wip-us.apache.org/repos/asf/curator/blob/05d241da/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java index 06285ca..cbbceac 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java @@ -1,10 +1,15 @@ package org.apache.curator.connection; +import com.google.common.base.Splitter; import org.apache.curator.CuratorZookeeperClient; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; import java.util.concurrent.Callable; public class StandardConnectionHandlingPolicy implements ConnectionHandlingPolicy { +private final Logger log = LoggerFactory.getLogger(getClass()); + @Override public boolean isEmulatingClassicHandling() { @@ -24,9 +29,15 @@ public class StandardConnectionHandlingPolicy implements ConnectionHandlingPolic @Override public PreRetryResult preRetry(CuratorZookeeperClient client) throws Exception { -// TODO - see if there are other servers to connect to if ( !client.isConnected() ) { +int serverCount = Splitter.on(",").omitEmptyStrings().splitToList(client.getCurrentConnectionString()).size(); +if ( serverCount > 1 ) +{ +log.info("Connection timed out and connection string is > 1. Resetting connection and trying again."); +client.reset(); // unfortunately, there's no way to guarantee that ZK tries a different server. Internally it calls Collections.shuffle(). Hopefully, this will result in a different server each time. +return PreRetryResult.WAIT_FOR_CONNECTION; +} return PreRetryResult.CONNECTION_TIMEOUT; }
[41/45] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-247
Merge branch 'CURATOR-3.0' into CURATOR-247 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/72f72237 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/72f72237 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/72f72237 Branch: refs/heads/CURATOR-248 Commit: 72f722372aedc221f6dd633113b5f9c30e8bd8e2 Parents: bc57786 d01eabe Author: randgalt Authored: Mon Aug 31 19:44:24 2015 -0700 Committer: randgalt Committed: Mon Aug 31 19:44:24 2015 -0700 -- .../org/apache/curator/retry/RetryForever.java | 60 .../java/org/apache/curator/TestRetryLoop.java | 20 +++ .../framework/recipes/cache/NodeCache.java | 41 ++--- curator-x-discovery-server/pom.xml | 6 ++ curator-x-discovery/pom.xml | 6 ++ curator-x-rpc/pom.xml | 6 ++ src/site/confluence/index.confluence| 7 +++ 7 files changed, 139 insertions(+), 7 deletions(-) --
[22/45] curator git commit: more work on repairing tests for new connection handling
more work on repairing tests for new connection handling Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/1a2a94b6 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/1a2a94b6 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/1a2a94b6 Branch: refs/heads/CURATOR-248 Commit: 1a2a94b625e7e1b5e535414e397e9b3a4173ca1b Parents: e001e00 Author: randgalt Authored: Sun Aug 23 10:54:29 2015 -0500 Committer: randgalt Committed: Sun Aug 23 10:54:29 2015 -0500 -- .../curator/framework/recipes/leader/TestLeaderSelectorEdges.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/1a2a94b6/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java -- diff --git a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java index 03708f1..0085968 100644 --- a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java +++ b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java @@ -158,7 +158,7 @@ public class TestLeaderSelectorEdges extends BaseClassForTests final CuratorFramework client = CuratorFrameworkFactory.builder() .connectString(server.getConnectString()) -.retryPolicy(new RetryNTimes(2, 1)) +.retryPolicy(new RetryNTimes(2, 100)) .connectionTimeoutMs(1000) .sessionTimeoutMs(6) .build();
[05/45] curator git commit: Continued work on new LOST behavior. Added some tests. To get correct behavior it's necessary to not retry connection failures. Retrying connection failures was never a goo
Continued work on new LOST behavior. Added some tests. To get correct behavior it's necessary to not retry connection failures. Retrying connection failures was never a good idea and here's a good opportunity to fix it as this requires client action to enable Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/62f3c33c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/62f3c33c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/62f3c33c Branch: refs/heads/CURATOR-248 Commit: 62f3c33cdb556eccf6fe1cc87ee74b3458431777 Parents: 2343daf Author: randgalt Authored: Fri Aug 21 17:35:44 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:35:44 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 24 ++--- .../apache/curator/CuratorZookeeperClient.java | 58 +++- .../main/java/org/apache/curator/RetryLoop.java | 12 +++ .../framework/CuratorFrameworkFactory.java | 2 +- .../framework/imps/CuratorFrameworkImpl.java| 43 - .../framework/state/ConnectionState.java| 5 + .../framework/state/ConnectionStateManager.java | 13 ++- .../framework/imps/TestBlockUntilConnected.java | 1 + .../imps/TestEnabledSessionExpiredState.java| 99 .../apache/curator/test/BaseClassForTests.java | 37 +++- .../java/org/apache/curator/test/Timing.java| 2 +- 11 files changed, 253 insertions(+), 43 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/62f3c33c/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index d3900a1..1dfdbef 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -171,6 +171,18 @@ class ConnectionState implements Watcher, Closeable return ensembleProvider; } +synchronized void reset() throws Exception +{ +log.debug("reset"); + +instanceIndex.incrementAndGet(); + +isConnected.set(false); +connectionStartMs = System.currentTimeMillis(); +zooKeeper.closeAndReset(); +zooKeeper.getZooKeeper(); // initiate connection +} + private synchronized void checkTimeouts() throws Exception { int minTimeout = Math.min(sessionTimeoutMs, connectionTimeoutMs); @@ -206,18 +218,6 @@ class ConnectionState implements Watcher, Closeable } } -private synchronized void reset() throws Exception -{ -log.debug("reset"); - -instanceIndex.incrementAndGet(); - -isConnected.set(false); -connectionStartMs = System.currentTimeMillis(); -zooKeeper.closeAndReset(); -zooKeeper.getZooKeeper(); // initiate connection -} - private boolean checkState(Event.KeeperState state, boolean wasConnected) { boolean isConnected = wasConnected; http://git-wip-us.apache.org/repos/asf/curator/blob/62f3c33c/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index 09b28b2..fbb2f4c 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -16,6 +16,7 @@ * specific language governing permissions and limitations * under the License. */ + package org.apache.curator; import com.google.common.base.Preconditions; @@ -43,12 +44,12 @@ import java.util.concurrent.atomic.AtomicReference; @SuppressWarnings("UnusedDeclaration") public class CuratorZookeeperClient implements Closeable { -private final Loggerlog = LoggerFactory.getLogger(getClass()); -private final ConnectionState state; -private final AtomicReference retryPolicy = new AtomicReference(); -private final int connectionTimeoutMs; -private final AtomicBoolean started = new AtomicBoolean(false); -private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); +private final Logger log = LoggerFactory.getLogger(getClass()); +private final ConnectionState state; +private final AtomicReference retryPolicy = new AtomicReference(); +private final int connectionTimeoutMs; +private final AtomicBoolean started =
[15/45] curator git commit: further refactoring. Abstracted old framework-level connection handling into ClassicInternalConnectionHandler. Probably more to do here
further refactoring. Abstracted old framework-level connection handling into ClassicInternalConnectionHandler. Probably more to do here Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/30bd7b65 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/30bd7b65 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/30bd7b65 Branch: refs/heads/CURATOR-248 Commit: 30bd7b655d201762d8ff74062964621879ac7134 Parents: e239137 Author: randgalt Authored: Sat Aug 22 19:29:36 2015 -0500 Committer: randgalt Committed: Sat Aug 22 19:29:36 2015 -0500 -- .../imps/ClassicInternalConnectionHandler.java | 58 ++ .../framework/imps/CuratorFrameworkImpl.java| 64 ++-- .../imps/InternalConnectionHandler.java | 10 +++ .../imps/StandardInternalConnectionHandler.java | 22 +++ .../framework/state/ConnectionStateManager.java | 8 ++- 5 files changed, 112 insertions(+), 50 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/30bd7b65/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java new file mode 100644 index 000..1de6e80 --- /dev/null +++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java @@ -0,0 +1,58 @@ +package org.apache.curator.framework.imps; + +import org.apache.curator.framework.state.ConnectionState; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +class ClassicInternalConnectionHandler implements InternalConnectionHandler +{ +private final Logger log = LoggerFactory.getLogger(getClass()); + +@Override +public void checkNewConnection(CuratorFrameworkImpl client) +{ +// NOP +} + +@Override +public boolean checkSessionExpirationEnabled() +{ +return false; +} + +@Override +public void suspendConnection(CuratorFrameworkImpl client) +{ +if ( client.setToSuspended() ) +{ +doSyncForSuspendedConnection(client, client.getZookeeperClient().getInstanceIndex()); +} +} + +private void doSyncForSuspendedConnection(final CuratorFrameworkImpl client, final long instanceIndex) +{ +// we appear to have disconnected, force a new ZK event and see if we can connect to another server +final BackgroundOperation operation = new BackgroundSyncImpl(client, null); +OperationAndData.ErrorCallback errorCallback = new OperationAndData.ErrorCallback() +{ +@Override +public void retriesExhausted(OperationAndData operationAndData) +{ +// if instanceIndex != newInstanceIndex, the ZooKeeper instance was reset/reallocated +// so the pending background sync is no longer valid. +// if instanceIndex is -1, this is the second try to sync - punt and mark the connection lost +if ( (instanceIndex < 0) || (instanceIndex == client.getZookeeperClient().getInstanceIndex()) ) +{ +client.addStateChange(ConnectionState.LOST); +} +else +{ +log.debug("suspendConnection() failure ignored as the ZooKeeper instance was reset. Retrying."); +// send -1 to signal that if it happens again, punt and mark the connection lost +doSyncForSuspendedConnection(client, -1); +} +} +}; +client.performBackgroundOperation(new OperationAndData(operation, "/", null, errorCallback, null)); +} +} http://git-wip-us.apache.org/repos/asf/curator/blob/30bd7b65/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java index 44a8ec6..b04987d 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java @@ -85,6 +85,7 @@ public class CuratorFrameworkImpl implements CuratorFramework private final NamespaceWatcherMap namespaceWatcherMap = new NamespaceWatcherMap(this); private final boolean useContainerParentsIfAva
[01/45] curator git commit: CURATOR-239 - Adding RetryForever retry policy
Repository: curator Updated Branches: refs/heads/CURATOR-248 d412f2320 -> c117b0853 CURATOR-239 - Adding RetryForever retry policy Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/6e8c1084 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/6e8c1084 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/6e8c1084 Branch: refs/heads/CURATOR-248 Commit: 6e8c10847d4b453a1a1aeafcb6c58f51889f2d1f Parents: 870b4d5 Author: Leandro Nunes Authored: Fri Jul 24 00:06:26 2015 +0100 Committer: Leandro Nunes Committed: Fri Jul 24 00:06:26 2015 +0100 -- .../org/apache/curator/retry/RetryForever.java | 58 .../java/org/apache/curator/TestRetryLoop.java | 20 +++ 2 files changed, 78 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/6e8c1084/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java -- diff --git a/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java b/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java new file mode 100644 index 000..27444b9 --- /dev/null +++ b/curator-client/src/main/java/org/apache/curator/retry/RetryForever.java @@ -0,0 +1,58 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.curator.retry; + +import org.apache.curator.RetryPolicy; +import org.apache.curator.RetrySleeper; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.util.concurrent.TimeUnit; + +import static com.google.common.base.Preconditions.checkArgument; + +/** + * {@link RetryPolicy} implementation that always allowsRetry. + */ +public class RetryForever implements RetryPolicy +{ +private static final Logger log = LoggerFactory.getLogger(RetryForever.class); + +private final int retryIntervalMs; + +public RetryForever(int retryIntervalMs) +{ +checkArgument(retryIntervalMs > 0); +this.retryIntervalMs = retryIntervalMs; +} + +@Override +public boolean allowRetry(int retryCount, long elapsedTimeMs, RetrySleeper sleeper) +{ +try +{ +sleeper.sleepFor(retryIntervalMs, TimeUnit.MILLISECONDS); +} +catch (InterruptedException e) +{ +log.warn("Error occurred while sleeping", e); +} +return true; +} +} http://git-wip-us.apache.org/repos/asf/curator/blob/6e8c1084/curator-client/src/test/java/org/apache/curator/TestRetryLoop.java -- diff --git a/curator-client/src/test/java/org/apache/curator/TestRetryLoop.java b/curator-client/src/test/java/org/apache/curator/TestRetryLoop.java index 0fa9020..17bb91e 100644 --- a/curator-client/src/test/java/org/apache/curator/TestRetryLoop.java +++ b/curator-client/src/test/java/org/apache/curator/TestRetryLoop.java @@ -19,14 +19,19 @@ package org.apache.curator; import org.apache.curator.retry.ExponentialBackoffRetry; +import org.apache.curator.retry.RetryForever; import org.apache.curator.retry.RetryOneTime; import org.apache.curator.test.BaseClassForTests; import org.apache.zookeeper.CreateMode; import org.apache.zookeeper.ZooDefs; +import org.mockito.Mockito; import org.testng.Assert; import org.testng.annotations.Test; + import java.util.concurrent.TimeUnit; +import static org.mockito.Mockito.times; + public class TestRetryLoop extends BaseClassForTests { @Test @@ -142,4 +147,19 @@ public class TestRetryLoop extends BaseClassForTests client.close(); } } + +@Test +public void testRetryForever() throws Exception +{ +int retryIntervalMs = 1; +RetrySleeper sleeper = Mockito.mock(RetrySleeper.class); +RetryForever retryForever = new RetryForever(retryIntervalMs); + +for (int i = 0; i < 10; i++) +{ +boolean allowed = retryForever.allowRetry(i, 0, sleeper); +Assert.assertTrue(al
[24/45] curator git commit: doc
doc Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/9c7cf5d8 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/9c7cf5d8 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/9c7cf5d8 Branch: refs/heads/CURATOR-248 Commit: 9c7cf5d8ba495bccdea2bcb6b377e95f5f99d521 Parents: 64d966c Author: randgalt Authored: Sun Aug 23 11:02:19 2015 -0500 Committer: randgalt Committed: Sun Aug 23 11:02:19 2015 -0500 -- .../apache/curator/framework/CuratorFrameworkFactory.java | 9 - 1 file changed, 4 insertions(+), 5 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/9c7cf5d8/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index 01a8666..c59d78f 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@ -368,11 +368,10 @@ public class CuratorFrameworkFactory * by the CuratorFramework instance. There should be no noticeable differences. * * - * Prior to 3.0.0, an elapsed connection timeout would be presented to the retry policy, possibly - * causing retries. Now, elapsed connection timeouts are only retried if there is an another server - * in the connection string. i.e. a new instance will be retried should the retry policy allow a retry. - * If no other servers remain, a {@link KeeperException.ConnectionLossException} is thrown immediately - * without notifying the retry policy. + * Prior to 3.0.0, each iteration of the retry policy would allow the connection timeout to elapse + * if the connection hadn't yet succeeded. This meant that the true connection timeout was the configured + * value times the maximum retries in the retry policy. This longstanding issue has been address. + * Now, the connection timeout can elapse only once for a single API call. * * * MOST IMPORTANTLY! Prior to 3.0.0, {@link ConnectionState#LOST} did not imply
[20/29] curator git commit: The true session timeout is a negotiated value between client and server. The new session timeout handling should use the negotiated value if available
The true session timeout is a negotiated value between client and server. The new session timeout handling should use the negotiated value if available Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/966b8dfc Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/966b8dfc Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/966b8dfc Branch: refs/heads/CURATOR-3.0 Commit: 966b8dfc8897f766f70a2334c90fa0913e06f996 Parents: 9c7cf5d Author: randgalt Authored: Sun Aug 23 20:31:21 2015 -0500 Committer: randgalt Committed: Sun Aug 23 20:31:21 2015 -0500 -- .../java/org/apache/curator/ConnectionState.java | 12 .../apache/curator/CuratorZookeeperClient.java | 10 ++ .../java/org/apache/curator/HandleHolder.java| 19 +++ .../framework/state/ConnectionStateManager.java | 5 +++-- 4 files changed, 44 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/966b8dfc/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index d6ddd33..555a52d 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -36,6 +36,7 @@ import java.util.Queue; import java.util.concurrent.Callable; import java.util.concurrent.ConcurrentLinkedQueue; import java.util.concurrent.atomic.AtomicBoolean; +import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicLong; import java.util.concurrent.atomic.AtomicReference; @@ -46,6 +47,7 @@ class ConnectionState implements Watcher, Closeable private final Logger log = LoggerFactory.getLogger(getClass()); private final HandleHolder zooKeeper; private final AtomicBoolean isConnected = new AtomicBoolean(false); +private final AtomicInteger lastNegotiatedSessionTimeoutMs = new AtomicInteger(0); private final EnsembleProvider ensembleProvider; private final int sessionTimeoutMs; private final int connectionTimeoutMs; @@ -141,6 +143,11 @@ class ConnectionState implements Watcher, Closeable return instanceIndex.get(); } +int getLastNegotiatedSessionTimeoutMs() +{ +return lastNegotiatedSessionTimeoutMs.get(); +} + @Override public void process(WatchedEvent event) { @@ -167,6 +174,11 @@ class ConnectionState implements Watcher, Closeable { isConnected.set(newIsConnected); connectionStartMs = System.currentTimeMillis(); +if ( newIsConnected ) +{ + lastNegotiatedSessionTimeoutMs.set(zooKeeper.getNegotiatedSessionTimeoutMs()); +log.debug("Negotiated session timeout: " + lastNegotiatedSessionTimeoutMs.get()); +} } } http://git-wip-us.apache.org/repos/asf/curator/blob/966b8dfc/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index d6c2072..471adf0 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -334,6 +334,16 @@ public class CuratorZookeeperClient implements Closeable return connectionHandlingPolicy; } +/** + * Return the most recent value of {@link ZooKeeper#getSessionTimeout()} or 0 + * + * @return session timeout or 0 + */ +public int getLastNegotiatedSessionTimeoutMs() +{ +return state.getLastNegotiatedSessionTimeoutMs(); +} + void addParentWatcher(Watcher watcher) { state.addParentWatcher(watcher); http://git-wip-us.apache.org/repos/asf/curator/blob/966b8dfc/curator-client/src/main/java/org/apache/curator/HandleHolder.java -- diff --git a/curator-client/src/main/java/org/apache/curator/HandleHolder.java b/curator-client/src/main/java/org/apache/curator/HandleHolder.java index 1f7cd91..8652f0c 100644 --- a/curator-client/src/main/java/org/apache/curator/HandleHolder.java +++ b/curator-client/src/main/java/org/apache/curator/HandleHolder.java @@ -39,6 +39,8 @@ class HandleHolder ZooKeeper getZooKeeper() throws Exception; String getConnectionString(); + +int getNegotiatedSess
[02/29] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-247
Merge branch 'CURATOR-3.0' into CURATOR-247 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/2343daf2 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/2343daf2 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/2343daf2 Branch: refs/heads/CURATOR-3.0 Commit: 2343daf29388566b0efa0b0a2ad21574fb534a27 Parents: 344634a 2fc9e37 Author: randgalt Authored: Fri Aug 21 15:11:59 2015 -0500 Committer: randgalt Committed: Fri Aug 21 15:11:59 2015 -0500 -- .../recipes/queue/DistributedIdQueue.java | 28 +++- .../recipes/queue/DistributedQueue.java | 7 - .../recipes/queue/TestDistributedIdQueue.java | 2 +- .../recipes/queue/TestDistributedQueue.java | 2 +- 4 files changed, 35 insertions(+), 4 deletions(-) --
[12/29] curator git commit: Curator has a big problem with thread interrupted states getting cleared. There are several issues on this (CURATOR-208, CURATOR-205, CURATOR-228, CURATOR-109
Curator has a big problem with thread interrupted states getting cleared. There are several issues on this (CURATOR-208, CURATOR-205, CURATOR-228, CURATOR-109 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/23554479 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/23554479 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/23554479 Branch: refs/heads/CURATOR-3.0 Commit: 23554479597d654fa8318cdc579fc3cc29bc2c54 Parents: 30bd7b6 Author: randgalt Authored: Sat Aug 22 20:10:34 2015 -0500 Committer: randgalt Committed: Sat Aug 22 20:10:34 2015 -0500 -- .../org/apache/curator/framework/state/ConnectionStateManager.java | 2 +- .../java/org/apache/curator/framework/imps/TestFrameworkEdges.java | 1 - 2 files changed, 1 insertion(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/23554479/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index 406099d..44d511b 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -248,7 +248,7 @@ public class ConnectionStateManager implements Closeable { try { -while ( !Thread.currentThread().isInterrupted() ) +while ( !Thread.currentThread().isInterrupted() && (state.get() == State.STARTED) ) { int pollMaxMs = (sessionTimeoutMs * 2) / 3; // 2/3 of session timeout final ConnectionState newState = eventQueue.poll(pollMaxMs, TimeUnit.MILLISECONDS); http://git-wip-us.apache.org/repos/asf/curator/blob/23554479/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java index 7407eab..15781dc 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java @@ -195,7 +195,6 @@ public class TestFrameworkEdges extends BaseClassForTests @Test public void testGetAclNoStat() throws Exception { - CuratorFramework client = CuratorFrameworkFactory.newClient(server.getConnectString(), timing.session(), timing.connection(), new RetryOneTime(1)); client.start(); try
[10/29] curator git commit: major refactoring. Abstracting old/new behavior into a pluggable ConnectionHandlingPolicy. Also, IMPORTANT, made the new behavior the default. This needs to be discussed bu
major refactoring. Abstracting old/new behavior into a pluggable ConnectionHandlingPolicy. Also, IMPORTANT, made the new behavior the default. This needs to be discussed but it's a major improvement and we should default to it. Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/e2391370 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/e2391370 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/e2391370 Branch: refs/heads/CURATOR-3.0 Commit: e239137019608f02cabb23c27ab13adcef88c027 Parents: 6381ccb Author: randgalt Authored: Sat Aug 22 19:06:55 2015 -0500 Committer: randgalt Committed: Sat Aug 22 19:06:55 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 85 .../apache/curator/CuratorZookeeperClient.java | 32 .../main/java/org/apache/curator/RetryLoop.java | 28 +-- .../ClassicConnectionHandlingPolicy.java| 48 +++ .../connection/ConnectionHandlingPolicy.java| 84 +++ .../StandardConnectionHandlingPolicy.java | 35 .../java/org/apache/curator/TestEnsurePath.java | 5 +- .../framework/CuratorFrameworkFactory.java | 54 +++-- .../framework/imps/CuratorFrameworkImpl.java| 43 +- .../framework/state/ConnectionState.java| 20 + .../framework/state/ConnectionStateManager.java | 9 +-- .../imps/TestEnabledSessionExpiredState.java| 5 +- .../apache/curator/test/BaseClassForTests.java | 6 +- 13 files changed, 336 insertions(+), 118 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/e2391370/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index c3d6921..d6ddd33 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -18,9 +18,10 @@ */ package org.apache.curator; -import org.apache.curator.utils.CloseableUtils; +import org.apache.curator.connection.ConnectionHandlingPolicy; import org.apache.curator.drivers.TracerDriver; import org.apache.curator.ensemble.EnsembleProvider; +import org.apache.curator.utils.CloseableUtils; import org.apache.curator.utils.DebugUtils; import org.apache.curator.utils.ZookeeperFactory; import org.apache.zookeeper.KeeperException; @@ -32,6 +33,7 @@ import org.slf4j.LoggerFactory; import java.io.Closeable; import java.io.IOException; import java.util.Queue; +import java.util.concurrent.Callable; import java.util.concurrent.ConcurrentLinkedQueue; import java.util.concurrent.atomic.AtomicBoolean; import java.util.concurrent.atomic.AtomicLong; @@ -48,18 +50,19 @@ class ConnectionState implements Watcher, Closeable private final int sessionTimeoutMs; private final int connectionTimeoutMs; private final AtomicReference tracer; +private final ConnectionHandlingPolicy connectionHandlingPolicy; private final Queue backgroundExceptions = new ConcurrentLinkedQueue(); private final Queue parentWatchers = new ConcurrentLinkedQueue(); private final AtomicLong instanceIndex = new AtomicLong(); private volatile long connectionStartMs = 0; -private final AtomicBoolean enableTimeoutChecks = new AtomicBoolean(true); -ConnectionState(ZookeeperFactory zookeeperFactory, EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher parentWatcher, AtomicReference tracer, boolean canBeReadOnly) +ConnectionState(ZookeeperFactory zookeeperFactory, EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher parentWatcher, AtomicReference tracer, boolean canBeReadOnly, ConnectionHandlingPolicy connectionHandlingPolicy) { this.ensembleProvider = ensembleProvider; this.sessionTimeoutMs = sessionTimeoutMs; this.connectionTimeoutMs = connectionTimeoutMs; this.tracer = tracer; +this.connectionHandlingPolicy = connectionHandlingPolicy; if ( parentWatcher != null ) { parentWatchers.offer(parentWatcher); @@ -68,11 +71,6 @@ class ConnectionState implements Watcher, Closeable zooKeeper = new HandleHolder(zookeeperFactory, this, ensembleProvider, sessionTimeoutMs, canBeReadOnly); } -void disableTimeoutChecks() -{ -enableTimeoutChecks.set(false); -} - ZooKeeper getZooKeeper() throws Exception { if ( SessionFailRetryLoop.sessionForThreadHasFailed() ) @@ -87,13 +85,10 @@ class ConnectionState implements Watcher, Closeable
[05/29] curator git commit: more tests
more tests Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/d3170099 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/d3170099 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/d3170099 Branch: refs/heads/CURATOR-3.0 Commit: d3170099757c7e17ff8fbee0c37d620aacb60d65 Parents: c5a4921 Author: randgalt Authored: Fri Aug 21 17:49:55 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:49:55 2015 -0500 -- .../imps/TestEnabledSessionExpiredState.java| 24 1 file changed, 24 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/d3170099/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java index 4c5ff81..150eb50 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java @@ -109,6 +109,30 @@ public class TestEnabledSessionExpiredState extends BaseClassForTests Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.RECONNECTED); } +@Test +public void testSessionExpirationFromTimeout() throws Exception +{ +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED); +server.stop(); +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.SUSPENDED); +Assert.assertEquals(states.poll(timing.multiple(2).session(), TimeUnit.MILLISECONDS), ConnectionState.LOST); +} + +@Test +public void testSessionExpirationFromTimeoutWithRestart() throws Exception +{ +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED); +server.stop(); +Thread.sleep(timing.multiple(1.2).session()); +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.SUSPENDED); +Assert.assertEquals(states.poll(timing.multiple(2).session(), TimeUnit.MILLISECONDS), ConnectionState.LOST); +server.restart(); +client.checkExists().forPath("/"); +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.RECONNECTED); + +Assert.assertNull(states.poll(timing.multiple(.5).milliseconds(), TimeUnit.MILLISECONDS)); // there should be no other events +} + @Override protected boolean enabledSessionExpiredStateAware() {
[16/29] curator git commit: more work on repairing tests for new connection handling
more work on repairing tests for new connection handling Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/e001e009 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/e001e009 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/e001e009 Branch: refs/heads/CURATOR-3.0 Commit: e001e0098f64baa8e0b3b887507bc98972c775dc Parents: 5f094f8 Author: randgalt Authored: Sun Aug 23 09:33:46 2015 -0500 Committer: randgalt Committed: Sun Aug 23 09:33:46 2015 -0500 -- .../org/apache/curator/CuratorZookeeperClient.java| 14 -- .../src/main/java/org/apache/curator/RetryLoop.java | 6 -- .../curator/framework/imps/TestFrameworkEdges.java| 3 --- .../x/discovery/details/TestServiceDiscovery.java | 2 ++ 4 files changed, 2 insertions(+), 23 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/e001e009/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index c8a9936..d6c2072 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -19,7 +19,6 @@ package org.apache.curator; -import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Preconditions; import org.apache.curator.connection.ClassicConnectionHandlingPolicy; import org.apache.curator.connection.ConnectionHandlingPolicy; @@ -54,7 +53,6 @@ public class CuratorZookeeperClient implements Closeable private final AtomicBoolean started = new AtomicBoolean(false); private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); private final ConnectionHandlingPolicy connectionHandlingPolicy; -private final AtomicReference debugException = new AtomicReference<>(); /** * @@ -336,18 +334,6 @@ public class CuratorZookeeperClient implements Closeable return connectionHandlingPolicy; } -@VisibleForTesting -public void setDebugException(Exception e) -{ -debugException.set(e); -} - -@VisibleForTesting -Exception getDebugException() -{ -return debugException.get(); -} - void addParentWatcher(Watcher watcher) { state.addParentWatcher(watcher); http://git-wip-us.apache.org/repos/asf/curator/blob/e001e009/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index 4353c61..e58ce20 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -96,12 +96,6 @@ public class RetryLoop */ public static T callWithRetry(CuratorZookeeperClient client, Callable proc) throws Exception { -Exception debugException = client.getDebugException(); -if ( debugException != null ) -{ -throw debugException; -} - return client.getConnectionHandlingPolicy().callWithRetry(client, proc); } http://git-wip-us.apache.org/repos/asf/curator/blob/e001e009/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java index 15781dc..ef77087 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java @@ -423,7 +423,6 @@ public class TestFrameworkEdges extends BaseClassForTests { throw new Error(e); } -client.getZookeeperClient().setDebugException(null); } try { @@ -441,7 +440,6 @@ public class TestFrameworkEdges extends BaseClassForTests server.stop(); // test foreground retry -client.getZookeeperClient().setDebugException(new KeeperException.ConnectionLossException()); client.checkExists().forPath("/hey"); Assert.assertTrue(semaphore.tryAcquire(MAX_RETRIES,
[29/29] curator git commit: Merge branch 'CURATOR-247' into CURATOR-3.0
Merge branch 'CURATOR-247' into CURATOR-3.0 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/d57aaeb7 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/d57aaeb7 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/d57aaeb7 Branch: refs/heads/CURATOR-3.0 Commit: d57aaeb77b40fd6a4f5aa52e3d4e44dbe02c1700 Parents: 667c97e c641c02 Author: randgalt Authored: Tue Sep 1 06:02:08 2015 -0700 Committer: randgalt Committed: Tue Sep 1 06:02:08 2015 -0700 -- .../org/apache/curator/ConnectionState.java | 108 +++ .../apache/curator/CuratorZookeeperClient.java | 112 +--- .../java/org/apache/curator/HandleHolder.java | 19 ++ .../main/java/org/apache/curator/RetryLoop.java | 18 +- .../ClassicConnectionHandlingPolicy.java| 68 +++ .../connection/ConnectionHandlingPolicy.java| 72 .../StandardConnectionHandlingPolicy.java | 56 ++ .../org/apache/curator/utils/DebugUtils.java| 11 +- .../java/org/apache/curator/TestEnsurePath.java | 3 + .../framework/CuratorFrameworkFactory.java | 58 ++ .../framework/api/UnhandledErrorListener.java | 4 +- .../imps/ClassicInternalConnectionHandler.java | 58 ++ .../framework/imps/CuratorFrameworkImpl.java| 86 - .../imps/InternalConnectionHandler.java | 10 ++ .../imps/StandardInternalConnectionHandler.java | 22 +++ .../framework/state/ConnectionState.java| 27 ++- .../framework/state/ConnectionStateManager.java | 91 +++--- .../framework/imps/TestBlockUntilConnected.java | 1 + .../imps/TestEnabledSessionExpiredState.java| 179 +++ .../framework/imps/TestFrameworkEdges.java | 2 - ...estResetConnectionWithBackgroundFailure.java | 19 +- .../recipes/leader/TestLeaderSelectorEdges.java | 6 +- .../locks/TestInterProcessMutexBase.java| 19 +- .../apache/curator/test/BaseClassForTests.java | 37 +++- .../java/org/apache/curator/test/Timing.java| 35 +++- .../discovery/details/TestServiceDiscovery.java | 2 + src/site/confluence/errors.confluence | 6 +- 27 files changed, 945 insertions(+), 184 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/d57aaeb7/curator-framework/src/test/java/org/apache/curator/framework/imps/TestFrameworkEdges.java --
[18/29] curator git commit: doc
doc Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/64d966c1 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/64d966c1 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/64d966c1 Branch: refs/heads/CURATOR-3.0 Commit: 64d966c18b9d18c40e13fda98e52d9253b281086 Parents: 1a2a94b Author: randgalt Authored: Sun Aug 23 10:57:48 2015 -0500 Committer: randgalt Committed: Sun Aug 23 10:57:48 2015 -0500 -- .../ClassicConnectionHandlingPolicy.java| 3 +++ .../connection/ConnectionHandlingPolicy.java| 20 ++-- .../StandardConnectionHandlingPolicy.java | 5 + 3 files changed, 22 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/64d966c1/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java index d0db0bb..1d53728 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ClassicConnectionHandlingPolicy.java @@ -4,6 +4,9 @@ import org.apache.curator.CuratorZookeeperClient; import org.apache.curator.RetryLoop; import java.util.concurrent.Callable; +/** + * Emulates the pre 3.0.0 Curator connection handling + */ public class ClassicConnectionHandlingPolicy implements ConnectionHandlingPolicy { @Override http://git-wip-us.apache.org/repos/asf/curator/blob/64d966c1/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java index 51bdccc..6cea67d 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java @@ -1,8 +1,13 @@ package org.apache.curator.connection; import org.apache.curator.CuratorZookeeperClient; +import org.apache.curator.RetryLoop; import java.util.concurrent.Callable; +/** + * Abstracts connection handling so that Curator can emulate it's old, pre 3.0.0 + * handling and update to newer handling. + */ public interface ConnectionHandlingPolicy { /** @@ -12,6 +17,15 @@ public interface ConnectionHandlingPolicy */ boolean isEmulatingClassicHandling(); +/** + * Called by {@link RetryLoop#callWithRetry(CuratorZookeeperClient, Callable)} to do the work + * of retrying + * + * @param client client + * @param proc the procedure to retry + * @return result + * @throws Exception errors + */ T callWithRetry(CuratorZookeeperClient client, Callable proc) throws Exception; enum CheckTimeoutsResult @@ -55,10 +69,4 @@ public interface ConnectionHandlingPolicy * @throws Exception errors */ CheckTimeoutsResult checkTimeouts(Callable hasNewConnectionString, long connectionStartMs, int sessionTimeoutMs, int connectionTimeoutMs) throws Exception; - -/* -int getDefaultConnectionTimeoutMs(); - -int getDefaultSessionTimeoutMs(); -*/ } http://git-wip-us.apache.org/repos/asf/curator/blob/64d966c1/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java index b16cd53..6906284 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java @@ -6,6 +6,11 @@ import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.util.concurrent.Callable; +/** + * Curator's standard connection handling since 3.0.0 + * + * @since 3.0.0 + */ public class StandardConnectionHandlingPolicy implements ConnectionHandlingPolicy { private final Logger log = LoggerFactory.getLogger(getClass());
[26/29] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-247
Merge branch 'CURATOR-3.0' into CURATOR-247 Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/72f72237 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/72f72237 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/72f72237 Branch: refs/heads/CURATOR-3.0 Commit: 72f722372aedc221f6dd633113b5f9c30e8bd8e2 Parents: bc57786 d01eabe Author: randgalt Authored: Mon Aug 31 19:44:24 2015 -0700 Committer: randgalt Committed: Mon Aug 31 19:44:24 2015 -0700 -- .../org/apache/curator/retry/RetryForever.java | 60 .../java/org/apache/curator/TestRetryLoop.java | 20 +++ .../framework/recipes/cache/NodeCache.java | 41 ++--- curator-x-discovery-server/pom.xml | 6 ++ curator-x-discovery/pom.xml | 6 ++ curator-x-rpc/pom.xml | 6 ++ src/site/confluence/index.confluence| 7 +++ 7 files changed, 139 insertions(+), 7 deletions(-) --
[08/29] curator git commit: Fixed testRetry() for new LOST behavior
Fixed testRetry() for new LOST behavior Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/ec2f9bd5 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/ec2f9bd5 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/ec2f9bd5 Branch: refs/heads/CURATOR-3.0 Commit: ec2f9bd555d01b324bd5ef690b1036d98e1f3702 Parents: 847cc0d Author: randgalt Authored: Sat Aug 22 11:06:33 2015 -0500 Committer: randgalt Committed: Sat Aug 22 11:06:33 2015 -0500 -- .../org/apache/curator/CuratorZookeeperClient.java | 17 +++-- .../main/java/org/apache/curator/RetryLoop.java| 12 +++- .../java/org/apache/curator/utils/DebugUtils.java | 12 +++- .../curator/framework/imps/TestFrameworkEdges.java | 4 +++- .../org/apache/curator/test/BaseClassForTests.java | 5 + 5 files changed, 41 insertions(+), 9 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/ec2f9bd5/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index ce6e9d3..a065d78 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -19,6 +19,7 @@ package org.apache.curator; +import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Preconditions; import org.apache.curator.drivers.TracerDriver; import org.apache.curator.ensemble.EnsembleProvider; @@ -51,6 +52,7 @@ public class CuratorZookeeperClient implements Closeable private final AtomicBoolean started = new AtomicBoolean(false); private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); private final boolean manageTimeouts; +private final AtomicReference debugException = new AtomicReference<>(); /** * @@ -207,8 +209,7 @@ public class CuratorZookeeperClient implements Closeable if ( !started.compareAndSet(false, true) ) { -IllegalStateException ise = new IllegalStateException("Already started"); -throw ise; +throw new IllegalStateException("Already started"); } state.start(); @@ -337,6 +338,18 @@ public class CuratorZookeeperClient implements Closeable return manageTimeouts; } +@VisibleForTesting +public void setDebugException(Exception e) +{ +debugException.set(e); +} + +@VisibleForTesting +Exception getDebugException() +{ +return debugException.get(); +} + void addParentWatcher(Watcher watcher) { state.addParentWatcher(watcher); http://git-wip-us.apache.org/repos/asf/curator/blob/ec2f9bd5/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index 8d77cf7..f6abf21 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -74,6 +74,7 @@ public class RetryLoop unit.sleep(time); } }; +private static final boolean checkInjectedDebugExceptions = Boolean.getBoolean(DebugUtils.PROPERTY_CHECK_INJECTED_DEBUG_EXCEPTIONS); /** * Returns the default retry sleeper @@ -103,13 +104,22 @@ public class RetryLoop { try { +if ( checkInjectedDebugExceptions ) +{ +Exception debugException = client.getDebugException(); +if ( debugException != null ) +{ +throw debugException; +} +} + client.internalBlockUntilConnectedOrTimedOut(); if ( !client.isConnected() && !client.retryConnectionTimeouts() ) { connectionFailed = true; break; } - + result = proc.call(); retryLoop.markComplete(); } http://git-wip-us.apache.org/repos/asf/curator/blob/ec2f9bd5/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java -- diff --git a/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java b/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java
[14/29] curator git commit: Longer connection timeout needed
Longer connection timeout needed Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/face4034 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/face4034 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/face4034 Branch: refs/heads/CURATOR-3.0 Commit: face4034e9fdcc9ffdb394c7c1682add834a1e10 Parents: 05d241d Author: randgalt Authored: Sat Aug 22 21:54:24 2015 -0500 Committer: randgalt Committed: Sat Aug 22 21:54:24 2015 -0500 -- .../framework/recipes/leader/TestLeaderSelectorEdges.java| 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/face4034/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java -- diff --git a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java index d13fea7..03708f1 100644 --- a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java +++ b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java @@ -159,7 +159,7 @@ public class TestLeaderSelectorEdges extends BaseClassForTests CuratorFrameworkFactory.builder() .connectString(server.getConnectString()) .retryPolicy(new RetryNTimes(2, 1)) -.connectionTimeoutMs(100) +.connectionTimeoutMs(1000) .sessionTimeoutMs(6) .build(); final CountDownLatch latch = new CountDownLatch(1); @@ -211,7 +211,7 @@ public class TestLeaderSelectorEdges extends BaseClassForTests CuratorFrameworkFactory.builder() .connectString(server.getConnectString()) .retryPolicy(new RetryNTimes(0, 0)) -.connectionTimeoutMs(100) +.connectionTimeoutMs(1000) .sessionTimeoutMs(6) .build(); final CountDownLatch latch = new CountDownLatch(1);
[22/29] curator git commit: Instead of closing the ZK instance and setting LOST directly, use the new testing API in 3.5 injectSessionExpiration(). This causes the ZK instance to close and LOST to get
Instead of closing the ZK instance and setting LOST directly, use the new testing API in 3.5 injectSessionExpiration(). This causes the ZK instance to close and LOST to get posted but has the benefit of sending a session expiration to all watchers Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/81bab455 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/81bab455 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/81bab455 Branch: refs/heads/CURATOR-3.0 Commit: 81bab455cf259f3b45d1fa0f7e26d78127f06d61 Parents: dd78816 Author: randgalt Authored: Mon Aug 24 00:21:35 2015 -0500 Committer: randgalt Committed: Mon Aug 24 00:21:35 2015 -0500 -- .../framework/state/ConnectionStateManager.java | 10 --- .../imps/TestEnabledSessionExpiredState.java| 28 2 files changed, 34 insertions(+), 4 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/81bab455/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index de71264..daa33f6 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -296,16 +296,18 @@ public class ConnectionStateManager implements Closeable int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; if ( elapsedMs >= useSessionTimeoutMs ) { -log.warn(String.format("Session timeout has elapsed while SUSPENDED. Posting LOST event and resetting the connection. Elapsed ms: %d. Session Timeout ms: %d", elapsedMs, useSessionTimeoutMs)); +log.warn(String.format("Session timeout has elapsed while SUSPENDED. Injecting a session expiration. Elapsed ms: %d. Session Timeout ms: %d", elapsedMs, useSessionTimeoutMs)); try { -client.getZookeeperClient().reset(); +// LOL - this method was proposed by me (JZ) in 2013 for totally unrelated reasons +// it got added to ZK 3.5 and now does exactly what we need +// https://issues.apache.org/jira/browse/ZOOKEEPER-1730 + client.getZookeeperClient().getZooKeeper().getTestable().injectSessionExpiration(); } catch ( Exception e ) { -log.error("Could not reset the connection", e); +log.error("Could not inject session expiration", e); } -addStateChange(ConnectionState.LOST); } } } http://git-wip-us.apache.org/repos/asf/curator/blob/81bab455/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java index a41d581..eff899d 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java @@ -29,11 +29,14 @@ import org.apache.curator.test.KillSession; import org.apache.curator.test.Timing; import org.apache.curator.utils.CloseableUtils; import org.apache.zookeeper.KeeperException; +import org.apache.zookeeper.WatchedEvent; +import org.apache.zookeeper.Watcher; import org.testng.Assert; import org.testng.annotations.AfterMethod; import org.testng.annotations.BeforeMethod; import org.testng.annotations.Test; import java.util.concurrent.BlockingQueue; +import java.util.concurrent.CountDownLatch; import java.util.concurrent.TimeUnit; public class TestEnabledSessionExpiredState extends BaseClassForTests @@ -79,6 +82,31 @@ public class TestEnabledSessionExpiredState extends BaseClassForTests } @Test +public void testInjectedWatchedEvent() throws Exception +{ +Assert.assertEquals(states.poll(timing.milliseconds(), TimeUnit.MILLISECONDS), ConnectionState.CONNECTED); + +final CountDownLatch latch = new CountDownLatch(1); +Watcher watcher = new Watcher() +{ +@Override +
[09/29] curator git commit: removed some test code
removed some test code Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/6381ccb6 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/6381ccb6 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/6381ccb6 Branch: refs/heads/CURATOR-3.0 Commit: 6381ccb6536f4710248a50ae5d0313399bbfe858 Parents: ec2f9bd Author: randgalt Authored: Sat Aug 22 17:50:09 2015 -0500 Committer: randgalt Committed: Sat Aug 22 17:50:09 2015 -0500 -- .../src/main/java/org/apache/curator/RetryLoop.java | 10 +++--- .../main/java/org/apache/curator/utils/DebugUtils.java| 1 - .../java/org/apache/curator/test/BaseClassForTests.java | 5 - 3 files changed, 3 insertions(+), 13 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/6381ccb6/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index f6abf21..92291c1 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -74,7 +74,6 @@ public class RetryLoop unit.sleep(time); } }; -private static final boolean checkInjectedDebugExceptions = Boolean.getBoolean(DebugUtils.PROPERTY_CHECK_INJECTED_DEBUG_EXCEPTIONS); /** * Returns the default retry sleeper @@ -104,13 +103,10 @@ public class RetryLoop { try { -if ( checkInjectedDebugExceptions ) +Exception debugException = client.getDebugException(); +if ( debugException != null ) { -Exception debugException = client.getDebugException(); -if ( debugException != null ) -{ -throw debugException; -} +throw debugException; } client.internalBlockUntilConnectedOrTimedOut(); http://git-wip-us.apache.org/repos/asf/curator/blob/6381ccb6/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java -- diff --git a/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java b/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java index 0e473fb..3e90600 100644 --- a/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java +++ b/curator-client/src/main/java/org/apache/curator/utils/DebugUtils.java @@ -26,7 +26,6 @@ public class DebugUtils public static final String PROPERTY_LOG_ONLY_FIRST_CONNECTION_ISSUE_AS_ERROR_LEVEL = "curator-log-only-first-connection-issue-as-error-level"; public static final String PROPERTY_REMOVE_WATCHERS_IN_FOREGROUND = "curator-remove-watchers-in-foreground"; public static final String PROPERTY_RETRY_FAILED_TESTS = "curator-retry-failed-tests"; -public static final String PROPERTY_CHECK_INJECTED_DEBUG_EXCEPTIONS = "curator-check-injected-debug-exceptions"; private DebugUtils() { http://git-wip-us.apache.org/repos/asf/curator/blob/6381ccb6/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java -- diff --git a/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java b/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java index 55dcb61..c9f3524 100644 --- a/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java +++ b/curator-test/src/main/java/org/apache/curator/test/BaseClassForTests.java @@ -44,20 +44,17 @@ public class BaseClassForTests private static final String INTERNAL_PROPERTY_DONT_LOG_CONNECTION_ISSUES; private static final String INTERNAL_PROPERTY_REMOVE_WATCHERS_IN_FOREGROUND; private static final String INTERNAL_RETRY_FAILED_TESTS; -private static final String INTERNAL_CHECK_INJECTED_DEBUG_EXCEPTIONS; static { String logConnectionIssues = null; String retryFailedTests = null; -String checkInjectedDebugExceptions = null; try { // use reflection to avoid adding a circular dependency in the pom Class debugUtilsClazz = Class.forName("org.apache.curator.utils.DebugUtils"); logConnectionIssues = (String)debugUtilsClazz.getField("PROPERTY_DONT_LOG_CONNECTION_ISSUES").get(null); retryFailedTests = (String)debugUtilsClazz.getField("PROPERTY_RETRY_FAILED_TESTS").get(null); -checkInjectedDebugExceptions = (String)debugUtilsClazz.getFie
[25/29] curator git commit: minor typo
minor typo Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/bc57786c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/bc57786c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/bc57786c Branch: refs/heads/CURATOR-3.0 Commit: bc57786c9c55f06f8918e0e2255277bfb2fbbecb Parents: b8240a1 Author: randgalt Authored: Mon Aug 24 11:11:16 2015 -0500 Committer: randgalt Committed: Mon Aug 24 11:11:16 2015 -0500 -- src/site/confluence/errors.confluence | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/bc57786c/src/site/confluence/errors.confluence -- diff --git a/src/site/confluence/errors.confluence b/src/site/confluence/errors.confluence index 0111cf4..a744128 100644 --- a/src/site/confluence/errors.confluence +++ b/src/site/confluence/errors.confluence @@ -26,7 +26,7 @@ appropriate action. These are the possible state changes: |READ_ONLY|The connection has gone into read\-only mode. This can only happen if you pass true for CuratorFrameworkFactory.Builder.canBeReadOnly(). See the ZooKeeper doc regarding read only connections: [[http://wiki.apache.org/hadoop/ZooKeeper/GSoCReadOnlyMode]]. The connection will remain in read only mode until another state change is sent.| |SUSPENDED|There has been a loss of connection. Leaders, locks, etc. should suspend until the connection is re\-established.| |RECONNECTED|A suspended or lost connection has been re\-established.| -|LOST|Curator will set the LOST state when it believes that the ZooKeeper session has expired. ZooKeeper connections have a session. When the session expires, clients must take appropriate action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper connection. Curator will set the LOST state when any of the following occurs: a) ZooKeeper returns a Watcher.Event.KeeperState.Expired or KeeperException.Code.SESSIONEXPIRED; b) Curator closes the internally managed ZooKeeper instance; c) The session timeout elapses during a network partition. It is possible to get a RECONNECTED state after this but you should still consider any locks, etc. as dirty/unstable. *NOTE* The meaning of LOST has changed since Curator 3.0.0. Prior to 3.0.0 LOST only meant that the retry policy had expired.| +|LOST|Curator will set the LOST state when it believes that the ZooKeeper session has expired. ZooKeeper connections have a session. When the session expires, clients must take appropriate action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper connection. Curator will set the LOST state when any of the following occurs: a) ZooKeeper returns a Watcher.Event.KeeperState.Expired or KeeperException.Code.SESSIONEXPIRED; b) Curator closes the internally managed ZooKeeper instance; c) The session timeout elapses during a network partition. It is possible to get a RECONNECTED state after this but you should still consider any locks, etc. as dirty/unstable. *NOTE*: The meaning of LOST has changed since Curator 3.0.0. Prior to 3.0.0 LOST only meant that the retry policy had expired.| {{UnhandledErrorListener}} is called when a background task, etc. catches an exception. In general, Curator users shouldn't care about these as they are logged. However, you can listen for them if you choose.
[17/29] curator git commit: more work on repairing tests for new connection handling
more work on repairing tests for new connection handling Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/1a2a94b6 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/1a2a94b6 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/1a2a94b6 Branch: refs/heads/CURATOR-3.0 Commit: 1a2a94b625e7e1b5e535414e397e9b3a4173ca1b Parents: e001e00 Author: randgalt Authored: Sun Aug 23 10:54:29 2015 -0500 Committer: randgalt Committed: Sun Aug 23 10:54:29 2015 -0500 -- .../curator/framework/recipes/leader/TestLeaderSelectorEdges.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/1a2a94b6/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java -- diff --git a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java index 03708f1..0085968 100644 --- a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java +++ b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/TestLeaderSelectorEdges.java @@ -158,7 +158,7 @@ public class TestLeaderSelectorEdges extends BaseClassForTests final CuratorFramework client = CuratorFrameworkFactory.builder() .connectString(server.getConnectString()) -.retryPolicy(new RetryNTimes(2, 1)) +.retryPolicy(new RetryNTimes(2, 100)) .connectionTimeoutMs(1000) .sessionTimeoutMs(6) .build();
[27/29] curator git commit: Use the negotiated session timeout in checkTimeouts()
Use the negotiated session timeout in checkTimeouts() Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/667b1d2c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/667b1d2c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/667b1d2c Branch: refs/heads/CURATOR-3.0 Commit: 667b1d2c7bd4b267585ae973dfff0f8b49f6b6ba Parents: 72f7223 Author: randgalt Authored: Mon Aug 31 19:48:12 2015 -0700 Committer: randgalt Committed: Mon Aug 31 19:48:12 2015 -0700 -- .../src/main/java/org/apache/curator/ConnectionState.java | 6 -- .../apache/curator/connection/ConnectionHandlingPolicy.java| 2 +- 2 files changed, 5 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/667b1d2c/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index f0996d8..eea2ce0 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -207,7 +207,9 @@ class ConnectionState implements Watcher, Closeable return zooKeeper.hasNewConnectionString(); } }; -ConnectionHandlingPolicy.CheckTimeoutsResult result = connectionHandlingPolicy.checkTimeouts(hasNewConnectionString, connectionStartMs, sessionTimeoutMs, connectionTimeoutMs); +int lastNegotiatedSessionTimeoutMs = getLastNegotiatedSessionTimeoutMs(); +int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; +ConnectionHandlingPolicy.CheckTimeoutsResult result = connectionHandlingPolicy.checkTimeouts(hasNewConnectionString, connectionStartMs, useSessionTimeoutMs, connectionTimeoutMs); switch ( result ) { default: @@ -227,7 +229,7 @@ class ConnectionState implements Watcher, Closeable if ( !Boolean.getBoolean(DebugUtils.PROPERTY_DONT_LOG_CONNECTION_ISSUES) ) { long elapsed = System.currentTimeMillis() - connectionStartMs; -int maxTimeout = Math.max(sessionTimeoutMs, connectionTimeoutMs); +int maxTimeout = Math.max(useSessionTimeoutMs, connectionTimeoutMs); log.warn(String.format("Connection attempt unsuccessful after %d (greater than max timeout of %d). Resetting connection and trying again with a new connection.", elapsed, maxTimeout)); } reset(); http://git-wip-us.apache.org/repos/asf/curator/blob/667b1d2c/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java index 6cea67d..9b6b895 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java @@ -63,7 +63,7 @@ public interface ConnectionHandlingPolicy * @param hasNewConnectionString proc to call to check if there is a new connection string. Important: the internal state is cleared after * this is called so you MUST handle the new connection string if true is returned * @param connectionStartMs the epoch/ms time that the connection was first initiated - * @param sessionTimeoutMs the configured session timeout in milliseconds + * @param sessionTimeoutMs the configured/negotiated session timeout in milliseconds * @param connectionTimeoutMs the configured connection timeout in milliseconds * @return result * @throws Exception errors
[21/29] curator git commit: code for useSessionTimeoutMs was wrong
code for useSessionTimeoutMs was wrong Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/dd788163 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/dd788163 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/dd788163 Branch: refs/heads/CURATOR-3.0 Commit: dd788163a278e2c10f71d2f0fd8b68137806f9f0 Parents: 966b8df Author: randgalt Authored: Sun Aug 23 20:33:55 2015 -0500 Committer: randgalt Committed: Sun Aug 23 20:33:55 2015 -0500 -- .../apache/curator/framework/state/ConnectionStateManager.java| 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/dd788163/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index 3da7534..de71264 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -292,7 +292,8 @@ public class ConnectionStateManager implements Closeable if ( (currentConnectionState == ConnectionState.SUSPENDED) && (startOfSuspendedEpoch != 0) ) { long elapsedMs = System.currentTimeMillis() - startOfSuspendedEpoch; -int useSessionTimeoutMs = Math.max(client.getZookeeperClient().getLastNegotiatedSessionTimeoutMs(), sessionTimeoutMs); +int lastNegotiatedSessionTimeoutMs = client.getZookeeperClient().getLastNegotiatedSessionTimeoutMs(); +int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; if ( elapsedMs >= useSessionTimeoutMs ) { log.warn(String.format("Session timeout has elapsed while SUSPENDED. Posting LOST event and resetting the connection. Elapsed ms: %d. Session Timeout ms: %d", elapsedMs, useSessionTimeoutMs));
[15/29] curator git commit: More refinement of classic/new connection handling. Reworked how the retry policy is invoked for each. New behavior is now confirmed to be: wait for connection timeout only
More refinement of classic/new connection handling. Reworked how the retry policy is invoked for each. New behavior is now confirmed to be: wait for connection timeout only once. Some tests will need work due to this Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/5f094f8b Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/5f094f8b Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/5f094f8b Branch: refs/heads/CURATOR-3.0 Commit: 5f094f8bb6dca3c056051cb8800b418839cca0e1 Parents: face403 Author: randgalt Authored: Sun Aug 23 07:49:17 2015 -0500 Committer: randgalt Committed: Sun Aug 23 07:49:17 2015 -0500 -- .../apache/curator/CuratorZookeeperClient.java | 7 ++- .../main/java/org/apache/curator/RetryLoop.java | 57 ++-- .../ClassicConnectionHandlingPolicy.java| 29 +++--- .../connection/ConnectionHandlingPolicy.java| 37 +++-- .../StandardConnectionHandlingPolicy.java | 39 -- .../imps/TestEnabledSessionExpiredState.java| 6 +-- ...estResetConnectionWithBackgroundFailure.java | 10 ++-- .../locks/TestInterProcessMutexBase.java| 19 +++ .../java/org/apache/curator/test/Timing.java| 36 - 9 files changed, 99 insertions(+), 141 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/5f094f8b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index 9342acf..c8a9936 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -358,7 +358,12 @@ public class CuratorZookeeperClient implements Closeable state.removeParentWatcher(watcher); } -void internalBlockUntilConnectedOrTimedOut() throws InterruptedException +/** + * For internal use only + * + * @throws InterruptedException interruptions + */ +public void internalBlockUntilConnectedOrTimedOut() throws InterruptedException { long waitTimeMs = connectionTimeoutMs; while ( !state.isConnected() && (waitTimeMs > 0) ) http://git-wip-us.apache.org/repos/asf/curator/blob/5f094f8b/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index a17cbf3..4353c61 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -96,62 +96,13 @@ public class RetryLoop */ public static T callWithRetry(CuratorZookeeperClient client, Callable proc) throws Exception { -T result = null; -RetryLoop retryLoop = client.newRetryLoop(); -boolean connectionFailed = false; -while ( retryLoop.shouldContinue() ) +Exception debugException = client.getDebugException(); +if ( debugException != null ) { -try -{ -Exception debugException = client.getDebugException(); -if ( debugException != null ) -{ -throw debugException; -} - -client.internalBlockUntilConnectedOrTimedOut(); - -switch ( client.getConnectionHandlingPolicy().preRetry(client) ) -{ -default: -case CALL_PROC: -{ -result = proc.call(); -retryLoop.markComplete(); -break; -} - -case WAIT_FOR_CONNECTION: -{ -break; // just loop -} - -case EXIT_RETRIES: -{ -retryLoop.markComplete(); -break; -} - -case CONNECTION_TIMEOUT: -{ -connectionFailed = true; -retryLoop.markComplete(); -break; -} -} -} -catch ( Exception e ) -{ -retryLoop.takeException(e); -} -} - -if ( connectionFailed ) -{ -throw new KeeperException.ConnectionLossEx
[19/29] curator git commit: doc
doc Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/9c7cf5d8 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/9c7cf5d8 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/9c7cf5d8 Branch: refs/heads/CURATOR-3.0 Commit: 9c7cf5d8ba495bccdea2bcb6b377e95f5f99d521 Parents: 64d966c Author: randgalt Authored: Sun Aug 23 11:02:19 2015 -0500 Committer: randgalt Committed: Sun Aug 23 11:02:19 2015 -0500 -- .../apache/curator/framework/CuratorFrameworkFactory.java | 9 - 1 file changed, 4 insertions(+), 5 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/9c7cf5d8/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index 01a8666..c59d78f 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@ -368,11 +368,10 @@ public class CuratorFrameworkFactory * by the CuratorFramework instance. There should be no noticeable differences. * * - * Prior to 3.0.0, an elapsed connection timeout would be presented to the retry policy, possibly - * causing retries. Now, elapsed connection timeouts are only retried if there is an another server - * in the connection string. i.e. a new instance will be retried should the retry policy allow a retry. - * If no other servers remain, a {@link KeeperException.ConnectionLossException} is thrown immediately - * without notifying the retry policy. + * Prior to 3.0.0, each iteration of the retry policy would allow the connection timeout to elapse + * if the connection hadn't yet succeeded. This meant that the true connection timeout was the configured + * value times the maximum retries in the retry policy. This longstanding issue has been address. + * Now, the connection timeout can elapse only once for a single API call. * * * MOST IMPORTANTLY! Prior to 3.0.0, {@link ConnectionState#LOST} did not imply
[03/29] curator git commit: Continued work on new LOST behavior. Added some tests. To get correct behavior it's necessary to not retry connection failures. Retrying connection failures was never a goo
Continued work on new LOST behavior. Added some tests. To get correct behavior it's necessary to not retry connection failures. Retrying connection failures was never a good idea and here's a good opportunity to fix it as this requires client action to enable Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/62f3c33c Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/62f3c33c Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/62f3c33c Branch: refs/heads/CURATOR-3.0 Commit: 62f3c33cdb556eccf6fe1cc87ee74b3458431777 Parents: 2343daf Author: randgalt Authored: Fri Aug 21 17:35:44 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:35:44 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 24 ++--- .../apache/curator/CuratorZookeeperClient.java | 58 +++- .../main/java/org/apache/curator/RetryLoop.java | 12 +++ .../framework/CuratorFrameworkFactory.java | 2 +- .../framework/imps/CuratorFrameworkImpl.java| 43 - .../framework/state/ConnectionState.java| 5 + .../framework/state/ConnectionStateManager.java | 13 ++- .../framework/imps/TestBlockUntilConnected.java | 1 + .../imps/TestEnabledSessionExpiredState.java| 99 .../apache/curator/test/BaseClassForTests.java | 37 +++- .../java/org/apache/curator/test/Timing.java| 2 +- 11 files changed, 253 insertions(+), 43 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/62f3c33c/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index d3900a1..1dfdbef 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -171,6 +171,18 @@ class ConnectionState implements Watcher, Closeable return ensembleProvider; } +synchronized void reset() throws Exception +{ +log.debug("reset"); + +instanceIndex.incrementAndGet(); + +isConnected.set(false); +connectionStartMs = System.currentTimeMillis(); +zooKeeper.closeAndReset(); +zooKeeper.getZooKeeper(); // initiate connection +} + private synchronized void checkTimeouts() throws Exception { int minTimeout = Math.min(sessionTimeoutMs, connectionTimeoutMs); @@ -206,18 +218,6 @@ class ConnectionState implements Watcher, Closeable } } -private synchronized void reset() throws Exception -{ -log.debug("reset"); - -instanceIndex.incrementAndGet(); - -isConnected.set(false); -connectionStartMs = System.currentTimeMillis(); -zooKeeper.closeAndReset(); -zooKeeper.getZooKeeper(); // initiate connection -} - private boolean checkState(Event.KeeperState state, boolean wasConnected) { boolean isConnected = wasConnected; http://git-wip-us.apache.org/repos/asf/curator/blob/62f3c33c/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index 09b28b2..fbb2f4c 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -16,6 +16,7 @@ * specific language governing permissions and limitations * under the License. */ + package org.apache.curator; import com.google.common.base.Preconditions; @@ -43,12 +44,12 @@ import java.util.concurrent.atomic.AtomicReference; @SuppressWarnings("UnusedDeclaration") public class CuratorZookeeperClient implements Closeable { -private final Loggerlog = LoggerFactory.getLogger(getClass()); -private final ConnectionState state; -private final AtomicReference retryPolicy = new AtomicReference(); -private final int connectionTimeoutMs; -private final AtomicBoolean started = new AtomicBoolean(false); -private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); +private final Logger log = LoggerFactory.getLogger(getClass()); +private final ConnectionState state; +private final AtomicReference retryPolicy = new AtomicReference(); +private final int connectionTimeoutMs; +private final AtomicBoolean started =
[23/29] curator git commit: Merge branch 'CURATOR-3.0' into CURATOR-247
Merge branch 'CURATOR-3.0' into CURATOR-247 Conflicts: curator-client/src/main/java/org/apache/curator/ConnectionState.java Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/1bbf301a Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/1bbf301a Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/1bbf301a Branch: refs/heads/CURATOR-3.0 Commit: 1bbf301ab75debf8866c78f13a32e65049a20772 Parents: 81bab45 f9af0ce Author: randgalt Authored: Mon Aug 24 08:02:08 2015 -0500 Committer: randgalt Committed: Mon Aug 24 08:02:08 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 36 +--- .../framework/imps/TestBlockUntilConnected.java | 24 + 2 files changed, 41 insertions(+), 19 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/1bbf301a/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --cc curator-client/src/main/java/org/apache/curator/ConnectionState.java index 555a52d,46ae9fd..f0996d8 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@@ -44,10 -41,9 +44,10 @@@ class ConnectionState implements Watche { private static final int MAX_BACKGROUND_EXCEPTIONS = 10; private static final boolean LOG_EVENTS = Boolean.getBoolean(DebugUtils.PROPERTY_LOG_EVENTS); - private final Logger log = LoggerFactory.getLogger(getClass()); + private static final Logger log = LoggerFactory.getLogger(ConnectionState.class); private final HandleHolder zooKeeper; private final AtomicBoolean isConnected = new AtomicBoolean(false); +private final AtomicInteger lastNegotiatedSessionTimeoutMs = new AtomicInteger(0); private final EnsembleProvider ensembleProvider; private final int sessionTimeoutMs; private final int connectionTimeoutMs; @@@ -156,29 -145,22 +156,27 @@@ log.debug("ConnectState watcher: " + event); } - for ( Watcher parentWatcher : parentWatchers ) - { - TimeTrace timeTrace = new TimeTrace("connection-state-parent-process", tracer.get()); - parentWatcher.process(event); - timeTrace.commit(); - } - - boolean wasConnected = isConnected.get(); - boolean newIsConnected = wasConnected; if ( event.getType() == Watcher.Event.EventType.None ) { - newIsConnected = checkState(event.getState(), wasConnected); + boolean wasConnected = isConnected.get(); + boolean newIsConnected = checkState(event.getState(), wasConnected); + if ( newIsConnected != wasConnected ) + { + isConnected.set(newIsConnected); + connectionStartMs = System.currentTimeMillis(); ++if ( newIsConnected ) ++{ ++ lastNegotiatedSessionTimeoutMs.set(zooKeeper.getNegotiatedSessionTimeoutMs()); ++log.debug("Negotiated session timeout: " + lastNegotiatedSessionTimeoutMs.get()); ++} + } } - if ( newIsConnected != wasConnected ) + for ( Watcher parentWatcher : parentWatchers ) { - isConnected.set(newIsConnected); - connectionStartMs = System.currentTimeMillis(); - if ( newIsConnected ) - { - lastNegotiatedSessionTimeoutMs.set(zooKeeper.getNegotiatedSessionTimeoutMs()); - log.debug("Negotiated session timeout: " + lastNegotiatedSessionTimeoutMs.get()); - } + TimeTrace timeTrace = new TimeTrace("connection-state-parent-process", tracer.get()); + parentWatcher.process(event); + timeTrace.commit(); } } http://git-wip-us.apache.org/repos/asf/curator/blob/1bbf301a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestBlockUntilConnected.java --
[07/29] curator git commit: wip
wip Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/847cc0d2 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/847cc0d2 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/847cc0d2 Branch: refs/heads/CURATOR-3.0 Commit: 847cc0d2415f59c2943d4a2734564119ffb38bb1 Parents: b8d4c3d Author: randgalt Authored: Sat Aug 22 10:47:01 2015 -0500 Committer: randgalt Committed: Sat Aug 22 10:47:01 2015 -0500 -- .../org/apache/curator/ConnectionState.java | 15 ++-- .../apache/curator/CuratorZookeeperClient.java | 36 ++-- .../framework/imps/CuratorFrameworkImpl.java| 12 ++- .../framework/state/ConnectionStateManager.java | 2 +- .../imps/TestEnabledSessionExpiredState.java| 2 +- ...estResetConnectionWithBackgroundFailure.java | 19 +++ .../java/org/apache/curator/test/Timing.java| 21 7 files changed, 84 insertions(+), 23 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/847cc0d2/curator-client/src/main/java/org/apache/curator/ConnectionState.java -- diff --git a/curator-client/src/main/java/org/apache/curator/ConnectionState.java b/curator-client/src/main/java/org/apache/curator/ConnectionState.java index 1dfdbef..c3d6921 100644 --- a/curator-client/src/main/java/org/apache/curator/ConnectionState.java +++ b/curator-client/src/main/java/org/apache/curator/ConnectionState.java @@ -52,6 +52,7 @@ class ConnectionState implements Watcher, Closeable private final Queue parentWatchers = new ConcurrentLinkedQueue(); private final AtomicLong instanceIndex = new AtomicLong(); private volatile long connectionStartMs = 0; +private final AtomicBoolean enableTimeoutChecks = new AtomicBoolean(true); ConnectionState(ZookeeperFactory zookeeperFactory, EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher parentWatcher, AtomicReference tracer, boolean canBeReadOnly) { @@ -67,6 +68,11 @@ class ConnectionState implements Watcher, Closeable zooKeeper = new HandleHolder(zookeeperFactory, this, ensembleProvider, sessionTimeoutMs, canBeReadOnly); } +void disableTimeoutChecks() +{ +enableTimeoutChecks.set(false); +} + ZooKeeper getZooKeeper() throws Exception { if ( SessionFailRetryLoop.sessionForThreadHasFailed() ) @@ -81,10 +87,13 @@ class ConnectionState implements Watcher, Closeable throw exception; } -boolean localIsConnected = isConnected.get(); -if ( !localIsConnected ) +if ( enableTimeoutChecks.get() ) { -checkTimeouts(); +boolean localIsConnected = isConnected.get(); +if ( !localIsConnected ) +{ +checkTimeouts(); +} } return zooKeeper.getZooKeeper(); http://git-wip-us.apache.org/repos/asf/curator/blob/847cc0d2/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java -- diff --git a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java index fbb2f4c..ce6e9d3 100644 --- a/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java +++ b/curator-client/src/main/java/org/apache/curator/CuratorZookeeperClient.java @@ -50,6 +50,7 @@ public class CuratorZookeeperClient implements Closeable private final int connectionTimeoutMs; private final AtomicBoolean started = new AtomicBoolean(false); private final AtomicReference tracer = new AtomicReference(new DefaultTracerDriver()); +private final boolean manageTimeouts; /** * @@ -61,7 +62,7 @@ public class CuratorZookeeperClient implements Closeable */ public CuratorZookeeperClient(String connectString, int sessionTimeoutMs, int connectionTimeoutMs, Watcher watcher, RetryPolicy retryPolicy) { -this(new DefaultZookeeperFactory(), new FixedEnsembleProvider(connectString), sessionTimeoutMs, connectionTimeoutMs, watcher, retryPolicy, false); +this(new DefaultZookeeperFactory(), new FixedEnsembleProvider(connectString), sessionTimeoutMs, connectionTimeoutMs, watcher, retryPolicy, false, true); } /** @@ -73,7 +74,7 @@ public class CuratorZookeeperClient implements Closeable */ public CuratorZookeeperClient(EnsembleProvider ensembleProvider, int sessionTimeoutMs, int connectionTimeoutMs, Watcher watcher, RetryPolicy retryPolicy) { -this(new DefaultZookeeperFactory(), ensembleProvider, sessionTimeoutMs, connectionTimeoutMs, watcher, retryP
[11/29] curator git commit: further refactoring. Abstracted old framework-level connection handling into ClassicInternalConnectionHandler. Probably more to do here
further refactoring. Abstracted old framework-level connection handling into ClassicInternalConnectionHandler. Probably more to do here Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/30bd7b65 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/30bd7b65 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/30bd7b65 Branch: refs/heads/CURATOR-3.0 Commit: 30bd7b655d201762d8ff74062964621879ac7134 Parents: e239137 Author: randgalt Authored: Sat Aug 22 19:29:36 2015 -0500 Committer: randgalt Committed: Sat Aug 22 19:29:36 2015 -0500 -- .../imps/ClassicInternalConnectionHandler.java | 58 ++ .../framework/imps/CuratorFrameworkImpl.java| 64 ++-- .../imps/InternalConnectionHandler.java | 10 +++ .../imps/StandardInternalConnectionHandler.java | 22 +++ .../framework/state/ConnectionStateManager.java | 8 ++- 5 files changed, 112 insertions(+), 50 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/30bd7b65/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java new file mode 100644 index 000..1de6e80 --- /dev/null +++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/ClassicInternalConnectionHandler.java @@ -0,0 +1,58 @@ +package org.apache.curator.framework.imps; + +import org.apache.curator.framework.state.ConnectionState; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +class ClassicInternalConnectionHandler implements InternalConnectionHandler +{ +private final Logger log = LoggerFactory.getLogger(getClass()); + +@Override +public void checkNewConnection(CuratorFrameworkImpl client) +{ +// NOP +} + +@Override +public boolean checkSessionExpirationEnabled() +{ +return false; +} + +@Override +public void suspendConnection(CuratorFrameworkImpl client) +{ +if ( client.setToSuspended() ) +{ +doSyncForSuspendedConnection(client, client.getZookeeperClient().getInstanceIndex()); +} +} + +private void doSyncForSuspendedConnection(final CuratorFrameworkImpl client, final long instanceIndex) +{ +// we appear to have disconnected, force a new ZK event and see if we can connect to another server +final BackgroundOperation operation = new BackgroundSyncImpl(client, null); +OperationAndData.ErrorCallback errorCallback = new OperationAndData.ErrorCallback() +{ +@Override +public void retriesExhausted(OperationAndData operationAndData) +{ +// if instanceIndex != newInstanceIndex, the ZooKeeper instance was reset/reallocated +// so the pending background sync is no longer valid. +// if instanceIndex is -1, this is the second try to sync - punt and mark the connection lost +if ( (instanceIndex < 0) || (instanceIndex == client.getZookeeperClient().getInstanceIndex()) ) +{ +client.addStateChange(ConnectionState.LOST); +} +else +{ +log.debug("suspendConnection() failure ignored as the ZooKeeper instance was reset. Retrying."); +// send -1 to signal that if it happens again, punt and mark the connection lost +doSyncForSuspendedConnection(client, -1); +} +} +}; +client.performBackgroundOperation(new OperationAndData(operation, "/", null, errorCallback, null)); +} +} http://git-wip-us.apache.org/repos/asf/curator/blob/30bd7b65/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java index 44a8ec6..b04987d 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java @@ -85,6 +85,7 @@ public class CuratorFrameworkImpl implements CuratorFramework private final NamespaceWatcherMap namespaceWatcherMap = new NamespaceWatcherMap(this); private final boolean useContainerParentsIfAva
[01/29] curator git commit: First pass at new (optional) definition of state LOST
Repository: curator Updated Branches: refs/heads/CURATOR-3.0 667c97e7f -> d57aaeb77 First pass at new (optional) definition of state LOST Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/344634ac Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/344634ac Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/344634ac Branch: refs/heads/CURATOR-3.0 Commit: 344634ac6e34e61bc0cc7b41923a1df4089c7948 Parents: 7d97259 Author: randgalt Authored: Fri Aug 21 12:10:24 2015 -0500 Committer: randgalt Committed: Fri Aug 21 12:10:24 2015 -0500 -- .../framework/CuratorFrameworkFactory.java | 19 + .../framework/api/UnhandledErrorListener.java | 4 +- .../framework/imps/CuratorFrameworkImpl.java| 10 ++- .../framework/state/ConnectionState.java| 35 +++-- .../framework/state/ConnectionStateManager.java | 75 +++- 5 files changed, 113 insertions(+), 30 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/344634ac/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index dcb2ee6..6209b06 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@ -31,6 +31,7 @@ import org.apache.curator.framework.imps.CuratorFrameworkImpl; import org.apache.curator.framework.imps.CuratorTempFrameworkImpl; import org.apache.curator.framework.imps.DefaultACLProvider; import org.apache.curator.framework.imps.GzipCompressionProvider; +import org.apache.curator.framework.state.ConnectionState; import org.apache.curator.utils.DefaultZookeeperFactory; import org.apache.curator.utils.ZookeeperFactory; import org.apache.zookeeper.CreateMode; @@ -116,6 +117,7 @@ public class CuratorFrameworkFactory private ACLProvider aclProvider = DEFAULT_ACL_PROVIDER; private boolean canBeReadOnly = false; private boolean useContainerParentsIfAvailable = true; +private boolean enableSessionExpiredState = false; /** * Apply the current values and build a new CuratorFramework @@ -343,6 +345,18 @@ public class CuratorFrameworkFactory return this; } +/** + * Changes the meaning of {@link ConnectionState#LOST} from it's pre Curator 3.0.0 meaning + * to a true lost session state. See the {@link ConnectionState#LOST} doc for details. + * + * @return this + */ +public Builder enableSessionExpiredState() +{ +this.enableSessionExpiredState = true; +return this; +} + public ACLProvider getAclProvider() { return aclProvider; @@ -398,6 +412,11 @@ public class CuratorFrameworkFactory return useContainerParentsIfAvailable; } +public boolean getEnableSessionExpiredState() +{ +return enableSessionExpiredState; +} + @Deprecated public String getAuthScheme() { http://git-wip-us.apache.org/repos/asf/curator/blob/344634ac/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java b/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java index b463af2..3721d4b 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/UnhandledErrorListener.java @@ -24,9 +24,7 @@ import org.apache.curator.framework.state.ConnectionStateListener; public interface UnhandledErrorListener { /** - * Called when an exception is caught in a background thread, handler, etc. Before this - * listener is called, the error will have been logged and a {@link ConnectionState#LOST} - * event will have been queued for any {@link ConnectionStateListener}s. + * Called when an exception is caught in a background thread, handler, etc. * * @param message Source message * @param e exception http://git-wip-us.apache.org/repos/asf/curator/blob/344634ac/curator-framework/src/main/java/org/apache/curator/framework/imps/CuratorFrameworkImpl.java --
[06/29] curator git commit: minor reformat
minor reformat Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/b8d4c3d7 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/b8d4c3d7 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/b8d4c3d7 Branch: refs/heads/CURATOR-3.0 Commit: b8d4c3d77de029917820634fa4ed21be19bbcf2c Parents: d317009 Author: randgalt Authored: Fri Aug 21 17:59:07 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:59:07 2015 -0500 -- .../src/test/java/org/apache/curator/TestEnsurePath.java | 2 ++ 1 file changed, 2 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/b8d4c3d7/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java -- diff --git a/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java b/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java index 4fe9561..871e4af 100644 --- a/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java +++ b/curator-client/src/test/java/org/apache/curator/TestEnsurePath.java @@ -51,6 +51,7 @@ public class TestEnsurePath CuratorZookeeperClient curator = mock(CuratorZookeeperClient.class); RetryPolicy retryPolicy = new RetryOneTime(1); RetryLoop retryLoop = new RetryLoop(retryPolicy, null); +when(curator.retryConnectionTimeouts()).thenReturn(true); when(curator.getZooKeeper()).thenReturn(client); when(curator.getRetryPolicy()).thenReturn(retryPolicy); when(curator.newRetryLoop()).thenReturn(retryLoop); @@ -76,6 +77,7 @@ public class TestEnsurePath RetryPolicy retryPolicy = new RetryOneTime(1); RetryLoop retryLoop = new RetryLoop(retryPolicy, null); final CuratorZookeeperClient curator = mock(CuratorZookeeperClient.class); +when(curator.retryConnectionTimeouts()).thenReturn(true); when(curator.getZooKeeper()).thenReturn(client); when(curator.getRetryPolicy()).thenReturn(retryPolicy); when(curator.newRetryLoop()).thenReturn(retryLoop);
[04/29] curator git commit: license
license Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/c5a49216 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/c5a49216 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/c5a49216 Branch: refs/heads/CURATOR-3.0 Commit: c5a49216cc78b05201a8ded357e50e0b6313 Parents: 62f3c33 Author: randgalt Authored: Fri Aug 21 17:37:15 2015 -0500 Committer: randgalt Committed: Fri Aug 21 17:37:15 2015 -0500 -- .../api/BackgroundPathableQuietlyable.java| 18 ++ .../imps/TestEnabledSessionExpiredState.java | 18 ++ 2 files changed, 36 insertions(+) -- http://git-wip-us.apache.org/repos/asf/curator/blob/c5a49216/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java index 8ed73fa..cd88739 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundPathableQuietlyable.java @@ -1,3 +1,21 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ package org.apache.curator.framework.api; public interface BackgroundPathableQuietlyable extends BackgroundPathable, Quietly> http://git-wip-us.apache.org/repos/asf/curator/blob/c5a49216/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java index 030a292..4c5ff81 100644 --- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java +++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java @@ -1,3 +1,21 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ package org.apache.curator.framework.imps; import com.google.common.collect.Queues;
[28/29] curator git commit: Use negotiated session timeout in processEvents()
Use negotiated session timeout in processEvents() Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/c641c029 Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/c641c029 Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/c641c029 Branch: refs/heads/CURATOR-3.0 Commit: c641c0296275bb12a156c2a6a1655e0cc968fad4 Parents: 667b1d2 Author: randgalt Authored: Mon Aug 31 19:50:21 2015 -0700 Committer: randgalt Committed: Mon Aug 31 19:50:21 2015 -0700 -- .../apache/curator/framework/state/ConnectionStateManager.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/c641c029/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java index daa33f6..3d44d45 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionStateManager.java @@ -250,7 +250,9 @@ public class ConnectionStateManager implements Closeable { while ( !Thread.currentThread().isInterrupted() && (state.get() == State.STARTED) ) { -int pollMaxMs = (sessionTimeoutMs * 2) / 3; // 2/3 of session timeout +int lastNegotiatedSessionTimeoutMs = client.getZookeeperClient().getLastNegotiatedSessionTimeoutMs(); +int useSessionTimeoutMs = (lastNegotiatedSessionTimeoutMs > 0) ? lastNegotiatedSessionTimeoutMs : sessionTimeoutMs; +int pollMaxMs = (useSessionTimeoutMs * 2) / 3; // 2/3 of session timeout final ConnectionState newState = eventQueue.poll(pollMaxMs, TimeUnit.MILLISECONDS); if ( newState != null ) {
[24/29] curator git commit: updated doc
updated doc Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/b8240a1d Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/b8240a1d Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/b8240a1d Branch: refs/heads/CURATOR-3.0 Commit: b8240a1dfcaf291196dc68fbfd1f2204c50951fc Parents: 1bbf301 Author: randgalt Authored: Mon Aug 24 09:28:53 2015 -0500 Committer: randgalt Committed: Mon Aug 24 09:28:53 2015 -0500 -- .../curator/framework/CuratorFrameworkFactory.java | 2 +- .../apache/curator/framework/state/ConnectionState.java | 7 --- .../framework/imps/TestEnabledSessionExpiredState.java | 11 +++ src/site/confluence/errors.confluence| 6 +++--- 4 files changed, 19 insertions(+), 7 deletions(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/b8240a1d/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java index c59d78f..daffa13 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFrameworkFactory.java @@ -381,7 +381,7 @@ public class CuratorFrameworkFactory * action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper * connection. Now, Curator will set the LOST state when any of the following occurs: * a) ZooKeeper returns a {@link Watcher.Event.KeeperState#Expired} or {@link KeeperException.Code#SESSIONEXPIRED}; - * b) Curator closes the internally managed ZooKeeper instance; c) The configured session timeout + * b) Curator closes the internally managed ZooKeeper instance; c) The session timeout * elapses during a network partition. * * http://git-wip-us.apache.org/repos/asf/curator/blob/b8240a1d/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java -- diff --git a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java index fe40abf..f9f245a 100644 --- a/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java +++ b/curator-framework/src/main/java/org/apache/curator/framework/state/ConnectionState.java @@ -18,6 +18,7 @@ */ package org.apache.curator.framework.state; +import org.apache.curator.connection.ConnectionHandlingPolicy; import org.apache.curator.framework.CuratorFrameworkFactory; import org.apache.zookeeper.KeeperException; import org.apache.zookeeper.Watcher; @@ -64,17 +65,17 @@ public enum ConnectionState /** * - * Curator will set the LOST state only when it believes that the ZooKeeper session + * Curator will set the LOST state when it believes that the ZooKeeper session * has expired. ZooKeeper connections have a session. When the session expires, clients must take appropriate * action. In Curator, this is complicated by the fact that Curator internally manages the ZooKeeper * connection. Curator will set the LOST state when any of the following occurs: * a) ZooKeeper returns a {@link Watcher.Event.KeeperState#Expired} or {@link KeeperException.Code#SESSIONEXPIRED}; - * b) Curator closes the internally managed ZooKeeper instance; c) The configured session timeout + * b) Curator closes the internally managed ZooKeeper instance; c) The session timeout * elapses during a network partition. * * * - * NOTE: see {@link CuratorFrameworkFactory.Builder#connectionHandlingPolicy} for an important note about a + * NOTE: see {@link CuratorFrameworkFactory.Builder#connectionHandlingPolicy(ConnectionHandlingPolicy)} for an important note about a * change in meaning to LOST since 3.0.0 * */ http://git-wip-us.apache.org/repos/asf/curator/blob/b8240a1d/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java -- diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabledSessionExpiredState.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestEnabl
[13/29] curator git commit: When the connection timeout elapses and there is more than one server in the connection string, reset the connection and try again
When the connection timeout elapses and there is more than one server in the connection string, reset the connection and try again Project: http://git-wip-us.apache.org/repos/asf/curator/repo Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/05d241da Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/05d241da Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/05d241da Branch: refs/heads/CURATOR-3.0 Commit: 05d241da642c6ba0d16b3ce97557128fad4dfe41 Parents: 2355447 Author: randgalt Authored: Sat Aug 22 20:32:41 2015 -0500 Committer: randgalt Committed: Sat Aug 22 20:32:41 2015 -0500 -- .../src/main/java/org/apache/curator/RetryLoop.java| 5 + .../curator/connection/ConnectionHandlingPolicy.java | 5 + .../connection/StandardConnectionHandlingPolicy.java | 13 - 3 files changed, 22 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/curator/blob/05d241da/curator-client/src/main/java/org/apache/curator/RetryLoop.java -- diff --git a/curator-client/src/main/java/org/apache/curator/RetryLoop.java b/curator-client/src/main/java/org/apache/curator/RetryLoop.java index 35d55a1..a17cbf3 100644 --- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java +++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java @@ -121,6 +121,11 @@ public class RetryLoop break; } +case WAIT_FOR_CONNECTION: +{ +break; // just loop +} + case EXIT_RETRIES: { retryLoop.markComplete(); http://git-wip-us.apache.org/repos/asf/curator/blob/05d241da/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java index f3ecce6..7f19159 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/ConnectionHandlingPolicy.java @@ -63,6 +63,11 @@ public interface ConnectionHandlingPolicy CALL_PROC, /** + * Wait again for connection success or timeout + */ +WAIT_FOR_CONNECTION, + +/** * Do not call the procedure and exit the retry loop */ EXIT_RETRIES, http://git-wip-us.apache.org/repos/asf/curator/blob/05d241da/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java -- diff --git a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java index 06285ca..cbbceac 100644 --- a/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java +++ b/curator-client/src/main/java/org/apache/curator/connection/StandardConnectionHandlingPolicy.java @@ -1,10 +1,15 @@ package org.apache.curator.connection; +import com.google.common.base.Splitter; import org.apache.curator.CuratorZookeeperClient; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; import java.util.concurrent.Callable; public class StandardConnectionHandlingPolicy implements ConnectionHandlingPolicy { +private final Logger log = LoggerFactory.getLogger(getClass()); + @Override public boolean isEmulatingClassicHandling() { @@ -24,9 +29,15 @@ public class StandardConnectionHandlingPolicy implements ConnectionHandlingPolic @Override public PreRetryResult preRetry(CuratorZookeeperClient client) throws Exception { -// TODO - see if there are other servers to connect to if ( !client.isConnected() ) { +int serverCount = Splitter.on(",").omitEmptyStrings().splitToList(client.getCurrentConnectionString()).size(); +if ( serverCount > 1 ) +{ +log.info("Connection timed out and connection string is > 1. Resetting connection and trying again."); +client.reset(); // unfortunately, there's no way to guarantee that ZK tries a different server. Internally it calls Collections.shuffle(). Hopefully, this will result in a different server each time. +return PreRetryResult.WAIT_FOR_CONNECTION; +} return PreRetryResult.CONNECTION_TIMEOUT; }