You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@curator.apache.org by ra...@apache.org on 2015/05/06 07:00:48 UTC
[1/8] curator git commit: Upgrade to ZooKeeper 3.5.0-alpha.
Repository: curator
Updated Branches:
refs/heads/CURATOR-160 36b9ed888 -> ea1a16841
Upgrade to ZooKeeper 3.5.0-alpha.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/271924bc
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/271924bc
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/271924bc
Branch: refs/heads/CURATOR-160
Commit: 271924bc42cb881688f8eba9d2313f091ac55241
Parents: 6a56c51
Author: Ioannis Canellos <io...@gmail.com>
Authored: Thu Nov 6 16:53:09 2014 +0200
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 12:58:20 2015 +0300
----------------------------------------------------------------------
.../framework/recipes/leader/ChaosMonkeyCnxnFactory.java | 7 -------
pom.xml | 2 +-
2 files changed, 1 insertion(+), 8 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/271924bc/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/ChaosMonkeyCnxnFactory.java
----------------------------------------------------------------------
diff --git a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/ChaosMonkeyCnxnFactory.java b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/ChaosMonkeyCnxnFactory.java
index 5f10c5e..ce5b23d 100644
--- a/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/ChaosMonkeyCnxnFactory.java
+++ b/curator-recipes/src/test/java/org/apache/curator/framework/recipes/leader/ChaosMonkeyCnxnFactory.java
@@ -59,13 +59,6 @@ public class ChaosMonkeyCnxnFactory extends NIOServerCnxnFactory
super.startup(new ChaosMonkeyZookeeperServer(zks));
}
- /**
- * Build a connection with a Chaos Monkey ZookeeperServer
- */
- protected NIOServerCnxn createConnection(SocketChannel sock, SelectionKey sk) throws IOException
- {
- return new NIOServerCnxn(zkServer, sock, sk, this);
- }
public static class ChaosMonkeyZookeeperServer extends ZooKeeperServer
{
http://git-wip-us.apache.org/repos/asf/curator/blob/271924bc/pom.xml
----------------------------------------------------------------------
diff --git a/pom.xml b/pom.xml
index be3ea33..51e86f6 100644
--- a/pom.xml
+++ b/pom.xml
@@ -82,7 +82,7 @@
<jetty-version>6.1.26</jetty-version>
<scannotation-version>1.0.2</scannotation-version>
<resteasy-jaxrs-version>2.3.0.GA</resteasy-jaxrs-version>
- <zookeeper-version>3.4.6</zookeeper-version>
+ <zookeeper-version>3.5.0-alpha</zookeeper-version>
<guava-version>16.0.1</guava-version>
<testng-version>6.8.8</testng-version>
<swift-version>0.12.0</swift-version>
[2/8] curator git commit: [CURATOR-160] Add EnsembleListener and
EnsembleTracker. Implement a DynamicEnsembleProvider. TestReconfiguration now
also tests the DynamicEnsembleProvider.
Posted by ra...@apache.org.
[CURATOR-160] Add EnsembleListener and EnsembleTracker. Implement a DynamicEnsembleProvider. TestReconfiguration now also tests the DynamicEnsembleProvider.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/59292d88
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/59292d88
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/59292d88
Branch: refs/heads/CURATOR-160
Commit: 59292d8854678d73d27a86746c1734f6c0f7e6a8
Parents: 3b5452f
Author: Ioannis Canellos <io...@gmail.com>
Authored: Tue Nov 11 16:35:57 2014 +0200
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 13:00:30 2015 +0300
----------------------------------------------------------------------
.../curator/ensemble/EnsembleListener.java | 24 +
.../dynamic/DynamicEnsembleProvider.java | 61 +++
.../curator/framework/imps/EnsembleTracker.java | 167 +++++++
.../framework/imps/TestReconfiguration.java | 489 +++++++++++--------
4 files changed, 538 insertions(+), 203 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/59292d88/curator-client/src/main/java/org/apache/curator/ensemble/EnsembleListener.java
----------------------------------------------------------------------
diff --git a/curator-client/src/main/java/org/apache/curator/ensemble/EnsembleListener.java b/curator-client/src/main/java/org/apache/curator/ensemble/EnsembleListener.java
new file mode 100644
index 0000000..8f963cd
--- /dev/null
+++ b/curator-client/src/main/java/org/apache/curator/ensemble/EnsembleListener.java
@@ -0,0 +1,24 @@
+/**
+ * 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.ensemble;
+
+public interface EnsembleListener {
+
+ void connectionStringUpdated(String connectionString);
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/59292d88/curator-client/src/main/java/org/apache/curator/ensemble/dynamic/DynamicEnsembleProvider.java
----------------------------------------------------------------------
diff --git a/curator-client/src/main/java/org/apache/curator/ensemble/dynamic/DynamicEnsembleProvider.java b/curator-client/src/main/java/org/apache/curator/ensemble/dynamic/DynamicEnsembleProvider.java
new file mode 100644
index 0000000..70b755f
--- /dev/null
+++ b/curator-client/src/main/java/org/apache/curator/ensemble/dynamic/DynamicEnsembleProvider.java
@@ -0,0 +1,61 @@
+/**
+ * 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.ensemble.dynamic;
+
+import com.google.common.base.Preconditions;
+import org.apache.curator.ensemble.EnsembleListener;
+import org.apache.curator.ensemble.EnsembleProvider;
+
+import java.io.IOException;
+import java.util.concurrent.atomic.AtomicReference;
+
+public class DynamicEnsembleProvider implements EnsembleProvider, EnsembleListener {
+
+ private final AtomicReference<String> connectionString = new AtomicReference<String>();
+
+ /**
+ * The connection string to use
+ *
+ * @param connectionString connection string
+ */
+ public DynamicEnsembleProvider(String connectionString)
+ {
+ this.connectionString.set(Preconditions.checkNotNull(connectionString, "connectionString cannot be null"));
+ }
+
+ @Override
+ public void start() throws Exception {
+ // NOP
+ }
+
+ @Override
+ public String getConnectionString() {
+ return connectionString.get();
+ }
+
+ @Override
+ public void close() throws IOException {
+ // NOP
+ }
+
+ @Override
+ public void connectionStringUpdated(String connectionString) {
+ this.connectionString.set(connectionString);
+ }
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/59292d88/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleTracker.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleTracker.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleTracker.java
new file mode 100644
index 0000000..a789e42
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleTracker.java
@@ -0,0 +1,167 @@
+/**
+ * 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.base.Function;
+import com.google.common.base.Preconditions;
+import org.apache.curator.ensemble.EnsembleListener;
+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.zookeeper.KeeperException;
+import org.apache.zookeeper.WatchedEvent;
+import org.apache.zookeeper.server.quorum.QuorumPeer;
+import org.apache.zookeeper.server.quorum.flexible.QuorumMaj;
+import org.apache.zookeeper.server.quorum.flexible.QuorumVerifier;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.ByteArrayInputStream;
+import java.io.Closeable;
+import java.io.IOException;
+import java.util.Properties;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicReference;
+
+/**
+ * Tracks changes to the ensemble and notifies registered {@link org.apache.curator.ensemble.EnsembleListener} instances.
+ */
+public class EnsembleTracker implements Closeable {
+
+ private final Logger log = LoggerFactory.getLogger(getClass());
+ private final CuratorFramework client;
+ private final AtomicReference<State> state = new AtomicReference<State>(State.LATENT);
+ private final ListenerContainer<EnsembleListener> listeners = new ListenerContainer<EnsembleListener>();
+ private final AtomicBoolean isConnected = new AtomicBoolean(true);
+ private final ConnectionStateListener connectionStateListener = new ConnectionStateListener() {
+ @Override
+ public void stateChanged(CuratorFramework client, ConnectionState newState) {
+ if ((newState == ConnectionState.CONNECTED) || (newState == ConnectionState.RECONNECTED)) {
+ if (isConnected.compareAndSet(false, true)) {
+ try {
+ reset();
+ } catch (Exception e) {
+ log.error("Trying to reset after reconnection", e);
+ }
+ }
+ } else {
+ isConnected.set(false);
+ }
+ }
+ };
+
+ private final CuratorWatcher watcher = new CuratorWatcher() {
+ @Override
+ public void process(WatchedEvent event) throws Exception {
+ reset();
+ }
+ };
+
+
+ private enum State {
+ LATENT,
+ STARTED,
+ CLOSED
+ }
+
+ private final BackgroundCallback backgroundCallback = new BackgroundCallback() {
+ @Override
+ public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
+ processBackgroundResult(event);
+ }
+ };
+
+
+ public EnsembleTracker(CuratorFramework client) {
+ this.client = client;
+ }
+
+ public void start() throws Exception {
+ Preconditions.checkState(state.compareAndSet(State.LATENT, State.STARTED), "Cannot be started more than once");
+ client.getConnectionStateListenable().addListener(connectionStateListener);
+ reset();
+ }
+
+ @Override
+ public void close() throws IOException {
+ if (state.compareAndSet(State.STARTED, State.CLOSED)) {
+ listeners.clear();
+ }
+ client.getConnectionStateListenable().removeListener(connectionStateListener);
+ }
+
+ /**
+ * Return the ensemble listenable
+ *
+ * @return listenable
+ */
+ public ListenerContainer<EnsembleListener> getListenable()
+ {
+ Preconditions.checkState(state.get() != State.CLOSED, "Closed");
+
+ return listeners;
+ }
+
+ private void reset() throws Exception {
+ client.getConfig().usingWatcher(watcher).inBackground(backgroundCallback).forEnsemble();
+ }
+
+ private void processBackgroundResult(CuratorEvent event) throws Exception {
+ switch (event.getType()) {
+ case GET_CONFIG: {
+ if (event.getResultCode() == KeeperException.Code.OK.intValue()) {
+ processConfigData(event.getData());
+ }
+ }
+ }
+ }
+
+ private void processConfigData(byte[] data) throws Exception {
+ Properties properties = new Properties();
+ properties.load(new ByteArrayInputStream(data));
+ QuorumVerifier qv = new QuorumMaj(properties);
+ StringBuilder sb = new StringBuilder();
+ for (QuorumPeer.QuorumServer server : qv.getAllMembers().values()) {
+ if (sb.length() != 0) {
+ sb.append(",");
+ }
+ sb.append(server.clientAddr.getAddress().getHostAddress()).append(":").append(server.clientAddr.getPort());
+ }
+
+ final String connectionString = sb.toString();
+ listeners.forEach
+ (
+ new Function<EnsembleListener, Void>() {
+ @Override
+ public Void apply(EnsembleListener listener) {
+ try {
+ listener.connectionStringUpdated(connectionString);
+ } catch (Exception e) {
+ log.error("Calling listener", e);
+ }
+ return null;
+ }
+ }
+ );
+ }
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/59292d88/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
index e8896ae..faec551 100644
--- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
+++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
@@ -18,6 +18,8 @@
*/
package org.apache.curator.framework.imps;
+import org.apache.curator.ensemble.EnsembleListener;
+import org.apache.curator.ensemble.dynamic.DynamicEnsembleProvider;
import org.apache.curator.framework.CuratorFramework;
import org.apache.curator.framework.CuratorFrameworkFactory;
import org.apache.curator.framework.api.BackgroundCallback;
@@ -35,246 +37,287 @@ import org.testng.annotations.Test;
import java.io.IOException;
import java.io.StringReader;
+import java.util.HashMap;
+import java.util.Map;
import java.util.Properties;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.TimeUnit;
+import java.util.concurrent.TimeoutException;
import java.util.concurrent.atomic.AtomicReference;
public class TestReconfiguration {
TestingCluster cluster;
+ DynamicEnsembleProvider dynamicEnsembleProvider;
+ WaitOnDelegateListener waitOnDelegateListener;
+ EnsembleTracker ensembleTracker;
+ CuratorFramework client;
+
+ String connectionString1to5;
+ String connectionString2to5;
+ String connectionString3to5;
@BeforeMethod
public void setup() throws Exception {
cluster = new TestingCluster(5);
cluster.start();
+
+ connectionString1to5 = cluster.getConnectString();
+ connectionString2to5 = getConnectionString(cluster, 2,3,4,5);
+ connectionString3to5 = getConnectionString(cluster, 3,4,5);
+
+ dynamicEnsembleProvider = new DynamicEnsembleProvider(connectionString1to5);
+ client = CuratorFrameworkFactory.builder()
+ .ensembleProvider(dynamicEnsembleProvider)
+ .retryPolicy(new RetryOneTime(1))
+ .build();
+ client.start();
+ client.blockUntilConnected();
+
+ //Wrap around the dynamic ensemble provider, so that we can wait until it has received the event.
+ waitOnDelegateListener = new WaitOnDelegateListener(dynamicEnsembleProvider);
+ ensembleTracker = new EnsembleTracker(client);
+ ensembleTracker.getListenable().addListener(waitOnDelegateListener);
+ ensembleTracker.start();
+ //Wait for the initial event.
+ waitOnDelegateListener.waitForEvent();
}
@AfterMethod
public void tearDown() throws IOException {
+ ensembleTracker.close();
+ client.close();
cluster.close();
}
@Test
public void testSyncIncremental() throws Exception {
- CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
- client.start();
- client.blockUntilConnected();
- try {
- Stat stat = new Stat();
- byte[] bytes = client.getConfig().storingStatIn(stat).forEnsemble();
- Assert.assertNotNull(bytes);
- QuorumVerifier qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- String server1 = getServerString(qv, cluster, 1L);
- String server2 = getServerString(qv, cluster, 2L);
-
- //Remove Servers
- bytes = client.reconfig().leaving("1").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- bytes = client.reconfig().leaving("2").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 3);
-
- //Add Servers
- bytes = client.reconfig().joining("server.1=" + server1).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- bytes = client.reconfig().joining("server.2=" + server2).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- } finally {
- client.close();
- }
+ Stat stat = new Stat();
+ byte[] bytes = client.getConfig().storingStatIn(stat).forEnsemble();
+ Assert.assertNotNull(bytes);
+ QuorumVerifier qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+
+ //Remove Servers
+ bytes = client.reconfig().leaving("1").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+
+ bytes = client.reconfig().leaving("2").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
+
+ //Add Servers
+ bytes = client.reconfig().joining("server.2=" + server2).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+
+ bytes = client.reconfig().joining("server.1=" + server1).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString1to5);
}
@Test
public void testAsyncIncremental() throws Exception {
- CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
- client.start();
- client.blockUntilConnected();
- try {
- final AtomicReference<byte[]> bytes = new AtomicReference<byte[]>();
- final BackgroundCallback callback = new BackgroundCallback() {
- @Override
- public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
- bytes.set(event.getData());
- ((CountDownLatch)event.getContext()).countDown();
+ final AtomicReference<byte[]> bytes = new AtomicReference<byte[]>();
+ final BackgroundCallback callback = new BackgroundCallback() {
+ @Override
+ public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
+ bytes.set(event.getData());
+ //We only need the latch on getConfig.
+ if (event.getContext() != null) {
+ ((CountDownLatch) event.getContext()).countDown();
}
+ }
- };
-
- CountDownLatch latch = new CountDownLatch(1);
- client.getConfig().inBackground(callback, latch).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- Assert.assertNotNull(bytes.get());
- QuorumVerifier qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- String server1 = getServerString(qv, cluster, 1L);
- String server2 = getServerString(qv, cluster, 2L);
-
-
- //Remove Servers
- latch = new CountDownLatch(1);
- client.reconfig().leaving("1").inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- latch = new CountDownLatch(1);
- client.reconfig().leaving("2").inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 3);
-
- //Add Servers
- latch = new CountDownLatch(1);
- client.reconfig().joining("server.1=" + server1).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- latch = new CountDownLatch(1);
- client.reconfig().joining("server.2=" + server2).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- } finally {
- client.close();
- }
+ };
+
+ CountDownLatch latch = new CountDownLatch(1);
+ client.getConfig().inBackground(callback, latch).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ Assert.assertNotNull(bytes.get());
+ QuorumVerifier qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+
+
+ //Remove Servers
+ client.reconfig().leaving("1").inBackground(callback).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ client.reconfig().leaving("2").inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ //Add Servers
+ client.reconfig().joining("server.2=" + server2).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+
+ client.reconfig().joining("server.1=" + server1).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString1to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
}
@Test
public void testSyncNonIncremental() throws Exception {
- CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
- client.start();
- client.blockUntilConnected();
- try {
- Stat stat = new Stat();
- byte[] bytes = client.getConfig().storingStatIn(stat).forEnsemble();
- Assert.assertNotNull(bytes);
- QuorumVerifier qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- String server1 = getServerString(qv, cluster, 1L);
- String server2 = getServerString(qv, cluster, 2L);
- String server3 = getServerString(qv, cluster, 3L);
- String server4 = getServerString(qv, cluster, 4L);
- String server5 = getServerString(qv, cluster, 5L);
-
- //Remove Servers
- bytes = client.reconfig()
- .withMembers("server.2=" + server2,
- "server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- bytes = client.reconfig()
- .withMembers("server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
-
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 3);
-
- //Add Servers
- bytes = client.reconfig()
- .withMembers("server.1=" + server1,
- "server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- bytes = client.reconfig()
- .withMembers("server.1=" + server1,
- "server.2=" + server2,
- "server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
- qv = getQuorumVerifier(bytes);
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- } finally {
- client.close();
- }
+ Stat stat = new Stat();
+ byte[] bytes = client.getConfig().storingStatIn(stat).forEnsemble();
+ Assert.assertNotNull(bytes);
+ QuorumVerifier qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+ String server3 = getServerString(qv, cluster, 3L);
+ String server4 = getServerString(qv, cluster, 4L);
+ String server5 = getServerString(qv, cluster, 5L);
+
+ //Remove Servers
+ bytes = client.reconfig()
+ .withMembers("server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+
+ bytes = client.reconfig()
+ .withMembers("server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
+
+ //Add Servers
+ bytes = client.reconfig()
+ .withMembers("server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+
+ bytes = client.reconfig()
+ .withMembers("server.1=" + server1,
+ "server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString1to5);
}
@Test
public void testAsyncNonIncremental() throws Exception {
- CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
- client.start();
- client.blockUntilConnected();
- try {
- final AtomicReference<byte[]> bytes = new AtomicReference<byte[]>();
- final BackgroundCallback callback = new BackgroundCallback() {
- @Override
- public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
- bytes.set(event.getData());
- ((CountDownLatch)event.getContext()).countDown();
- }
+ final AtomicReference<byte[]> bytes = new AtomicReference<byte[]>();
+ final BackgroundCallback callback = new BackgroundCallback() {
+ @Override
+ public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
+ bytes.set(event.getData());
+ ((CountDownLatch) event.getContext()).countDown();
+ }
- };
-
- CountDownLatch latch = new CountDownLatch(1);
- client.getConfig().inBackground(callback, latch).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- Assert.assertNotNull(bytes.get());
- QuorumVerifier qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- String server1 = getServerString(qv, cluster, 1L);
- String server2 = getServerString(qv, cluster, 2L);
- String server3 = getServerString(qv, cluster, 3L);
- String server4 = getServerString(qv, cluster, 4L);
- String server5 = getServerString(qv, cluster, 5L);
-
- //Remove Servers
- latch = new CountDownLatch(1);
- client.reconfig()
- .withMembers("server.2=" + server2,
- "server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- latch = new CountDownLatch(1);
- client.reconfig()
- .withMembers("server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 3);
-
- //Add Servers
- latch = new CountDownLatch(1);
- client.reconfig()
- .withMembers("server.1=" + server1,
- "server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 4);
- latch = new CountDownLatch(1);
- client.reconfig()
- .withMembers("server.1=" + server1,
- "server.2=" + server2,
- "server.3=" + server3,
- "server.4=" + server4,
- "server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
- latch.await(5, TimeUnit.SECONDS);
- qv = getQuorumVerifier(bytes.get());
- Assert.assertEquals(qv.getAllMembers().size(), 5);
- } finally {
- client.close();
- }
+ };
+
+ CountDownLatch latch = new CountDownLatch(1);
+ client.getConfig().inBackground(callback, latch).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ Assert.assertNotNull(bytes.get());
+ QuorumVerifier qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+ String server3 = getServerString(qv, cluster, 3L);
+ String server4 = getServerString(qv, cluster, 4L);
+ String server5 = getServerString(qv, cluster, 5L);
+
+ //Remove Servers
+ client.reconfig()
+ .withMembers("server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ client.reconfig()
+ .withMembers("server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ //Add Servers
+ client.reconfig()
+ .withMembers("server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+
+ client.reconfig()
+ .withMembers("server.1=" + server1,
+ "server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ waitOnDelegateListener.waitForEvent();
+ Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString1to5);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
}
@@ -302,4 +345,44 @@ public class TestReconfiguration {
return str + ";" + getInstance(cluster, (int) id).getConnectString();
}
}
+
+ static String getConnectionString(TestingCluster cluster, long... ids) throws Exception {
+ StringBuilder sb = new StringBuilder();
+ Map<Long, InstanceSpec> specs = new HashMap<Long, InstanceSpec>();
+ for (InstanceSpec spec : cluster.getInstances()) {
+ specs.put(new Long(spec.getServerId()), spec);
+ }
+ for (long id : ids) {
+ if (sb.length() != 0) {
+ sb.append(",");
+ }
+ sb.append(specs.get(id).getConnectString());
+ }
+ return sb.toString();
+ }
+
+ //Simple EnsembleListener that can wait until the delegate handles the event.
+ private static class WaitOnDelegateListener implements EnsembleListener {
+ private CountDownLatch latch = new CountDownLatch(1);
+
+ private final EnsembleListener delegate;
+
+ private WaitOnDelegateListener(EnsembleListener delegate) {
+ this.delegate = delegate;
+ }
+
+ @Override
+ public void connectionStringUpdated(String connectionString) {
+ delegate.connectionStringUpdated(connectionString);
+ latch.countDown();
+ }
+
+ public void waitForEvent() throws InterruptedException, TimeoutException {
+ if (latch.await(5, TimeUnit.SECONDS)) {
+ latch = new CountDownLatch(1);
+ } else {
+ throw new TimeoutException("Failed to receive event in time.");
+ }
+ }
+ };
}
\ No newline at end of file
[6/8] curator git commit: [CURATOR-160] Fix QuorumConfigBuilder to
provide a valid server string.
Posted by ra...@apache.org.
[CURATOR-160] Fix QuorumConfigBuilder to provide a valid server string.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/3b5452f2
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/3b5452f2
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/3b5452f2
Branch: refs/heads/CURATOR-160
Commit: 3b5452f2c8c9c72a8298dc85671e71c2add16604
Parents: d4a5c63
Author: Ioannis Canellos <io...@gmail.com>
Authored: Tue Nov 11 12:28:53 2014 +0200
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 13:00:30 2015 +0300
----------------------------------------------------------------------
.../src/main/java/org/apache/curator/test/QuorumConfigBuilder.java | 2 +-
1 file changed, 1 insertion(+), 1 deletion(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/3b5452f2/curator-test/src/main/java/org/apache/curator/test/QuorumConfigBuilder.java
----------------------------------------------------------------------
diff --git a/curator-test/src/main/java/org/apache/curator/test/QuorumConfigBuilder.java b/curator-test/src/main/java/org/apache/curator/test/QuorumConfigBuilder.java
index 8add08e..02979ee 100644
--- a/curator-test/src/main/java/org/apache/curator/test/QuorumConfigBuilder.java
+++ b/curator-test/src/main/java/org/apache/curator/test/QuorumConfigBuilder.java
@@ -99,7 +99,7 @@ public class QuorumConfigBuilder
{
for ( InstanceSpec thisSpec : instanceSpecs )
{
- properties.setProperty("server." + thisSpec.getServerId(), String.format("localhost:%d:%d", thisSpec.getQuorumPort(), thisSpec.getElectionPort()));
+ properties.setProperty("server." + thisSpec.getServerId(), String.format("localhost:%d:%d;localhost:%d", thisSpec.getQuorumPort(), thisSpec.getElectionPort(), thisSpec.getPort()));
}
}
[8/8] curator git commit: Merge branch 'CURATOR-160' of
github.com:iocanel/curator into CURATOR-160
Posted by ra...@apache.org.
Merge branch 'CURATOR-160' of github.com:iocanel/curator into CURATOR-160
Conflicts:
curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/ea1a1684
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/ea1a1684
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/ea1a1684
Branch: refs/heads/CURATOR-160
Commit: ea1a1684198ca2fa317486a881d5f48466fbf8f8
Parents: 36b9ed8 625441b
Author: randgalt <ra...@apache.org>
Authored: Wed May 6 00:00:37 2015 -0500
Committer: randgalt <ra...@apache.org>
Committed: Wed May 6 00:00:37 2015 -0500
----------------------------------------------------------------------
.../api/BackgroundStatConfigEnsembleable.java | 24 +++++
.../api/BackgroundStatEnsembleable.java | 24 +++++
.../curator/framework/api/Configurable.java | 2 +-
.../JoinBackgroundStatConfigEnsembleable.java | 30 ++++++
.../LeaveBackgroundStatConfigEnsembleable.java | 30 ++++++
.../curator/framework/api/ReconfigBuilder.java | 6 +-
.../framework/imps/CreateBuilderImpl.java | 14 +--
.../framework/imps/DeleteBuilderImpl.java | 103 +++++++++----------
.../framework/imps/GetConfigBuilderImpl.java | 2 +-
.../framework/imps/ReconfigBuilderImpl.java | 48 ++++-----
.../framework/imps/TestReconfiguration.java | 32 +++---
11 files changed, 204 insertions(+), 111 deletions(-)
----------------------------------------------------------------------
[7/8] curator git commit: [CURATOR-160] Fix watcher support in
GetConfigBuilderImpl.
Posted by ra...@apache.org.
[CURATOR-160] Fix watcher support in GetConfigBuilderImpl.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/d4a5c63f
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/d4a5c63f
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/d4a5c63f
Branch: refs/heads/CURATOR-160
Commit: d4a5c63fd291dfe1315537e6c76f442ae188e8a1
Parents: 0e92b56
Author: Ioannis Canellos <io...@gmail.com>
Authored: Tue Nov 11 12:28:10 2014 +0200
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 13:00:30 2015 +0300
----------------------------------------------------------------------
.../org/apache/curator/framework/imps/GetConfigBuilderImpl.java | 4 ++--
1 file changed, 2 insertions(+), 2 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/d4a5c63f/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
index 54b1862..59a621a 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
@@ -158,11 +158,11 @@ public class GetConfigBuilderImpl implements GetConfigBuilder, BackgroundOperati
};
if ( watching.isWatched() )
{
- client.getZooKeeper().getConfig(watching.getWatcher(), callback, backgrounding.getContext());
+ client.getZooKeeper().getConfig(false, callback, backgrounding.getContext());
}
else
{
- client.getZooKeeper().getConfig(false, callback, backgrounding.getContext());
+ client.getZooKeeper().getConfig(watching.getWatcher(), callback, backgrounding.getContext());
}
}
[3/8] curator git commit: Update project version to 3.0.0-SNAPSHOT.
Posted by ra...@apache.org.
Update project version to 3.0.0-SNAPSHOT.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/e11ef78a
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/e11ef78a
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/e11ef78a
Branch: refs/heads/CURATOR-160
Commit: e11ef78a604531783b5f39b0a9c0c65efef85f94
Parents: 271924b
Author: Ioannis Canellos <io...@gmail.com>
Authored: Thu Nov 6 17:30:29 2014 +0200
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 13:00:30 2015 +0300
----------------------------------------------------------------------
curator-client/pom.xml | 4 ++--
curator-examples/pom.xml | 2 +-
curator-framework/pom.xml | 4 ++--
curator-recipes/pom.xml | 4 ++--
curator-test/pom.xml | 4 ++--
curator-x-discovery-server/pom.xml | 4 ++--
curator-x-discovery/pom.xml | 4 ++--
curator-x-rpc/pom.xml | 4 ++--
pom.xml | 2 +-
9 files changed, 16 insertions(+), 16 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-client/pom.xml
----------------------------------------------------------------------
diff --git a/curator-client/pom.xml b/curator-client/pom.xml
index 251d2b0..0f3955b 100644
--- a/curator-client/pom.xml
+++ b/curator-client/pom.xml
@@ -24,11 +24,11 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-client</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<packaging>bundle</packaging>
<name>Curator Client</name>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-examples/pom.xml
----------------------------------------------------------------------
diff --git a/curator-examples/pom.xml b/curator-examples/pom.xml
index 96ea4b1..825ca16 100644
--- a/curator-examples/pom.xml
+++ b/curator-examples/pom.xml
@@ -24,7 +24,7 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-examples</artifactId>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-framework/pom.xml
----------------------------------------------------------------------
diff --git a/curator-framework/pom.xml b/curator-framework/pom.xml
index 67f514f..a45ce77 100644
--- a/curator-framework/pom.xml
+++ b/curator-framework/pom.xml
@@ -24,11 +24,11 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-framework</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<packaging>bundle</packaging>
<name>Curator Framework</name>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-recipes/pom.xml
----------------------------------------------------------------------
diff --git a/curator-recipes/pom.xml b/curator-recipes/pom.xml
index 3ce7d63..9d1d081 100644
--- a/curator-recipes/pom.xml
+++ b/curator-recipes/pom.xml
@@ -24,11 +24,11 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-recipes</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<packaging>bundle</packaging>
<name>Curator Recipes</name>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-test/pom.xml
----------------------------------------------------------------------
diff --git a/curator-test/pom.xml b/curator-test/pom.xml
index 153db7b..1cbd93c 100644
--- a/curator-test/pom.xml
+++ b/curator-test/pom.xml
@@ -24,11 +24,11 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-test</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<name>Curator Testing</name>
<description>Unit testing utilities.</description>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-x-discovery-server/pom.xml
----------------------------------------------------------------------
diff --git a/curator-x-discovery-server/pom.xml b/curator-x-discovery-server/pom.xml
index 6035681..352310f 100644
--- a/curator-x-discovery-server/pom.xml
+++ b/curator-x-discovery-server/pom.xml
@@ -24,11 +24,11 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-x-discovery-server</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<packaging>bundle</packaging>
<name>Curator Service Discovery Server</name>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-x-discovery/pom.xml
----------------------------------------------------------------------
diff --git a/curator-x-discovery/pom.xml b/curator-x-discovery/pom.xml
index b557211..e211d89 100644
--- a/curator-x-discovery/pom.xml
+++ b/curator-x-discovery/pom.xml
@@ -24,11 +24,11 @@
<parent>
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<artifactId>curator-x-discovery</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<packaging>bundle</packaging>
<name>Curator Service Discovery</name>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/curator-x-rpc/pom.xml
----------------------------------------------------------------------
diff --git a/curator-x-rpc/pom.xml b/curator-x-rpc/pom.xml
index 7dfcc85..b647294 100644
--- a/curator-x-rpc/pom.xml
+++ b/curator-x-rpc/pom.xml
@@ -22,12 +22,12 @@
<parent>
<artifactId>apache-curator</artifactId>
<groupId>org.apache.curator</groupId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
</parent>
<modelVersion>4.0.0</modelVersion>
<artifactId>curator-x-rpc</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<name>Curator RPC Proxy</name>
<description>A proxy that bridges non-java environments with the Curator framework and recipes</description>
http://git-wip-us.apache.org/repos/asf/curator/blob/e11ef78a/pom.xml
----------------------------------------------------------------------
diff --git a/pom.xml b/pom.xml
index 51e86f6..598a0ec 100644
--- a/pom.xml
+++ b/pom.xml
@@ -28,7 +28,7 @@
<groupId>org.apache.curator</groupId>
<artifactId>apache-curator</artifactId>
- <version>2.7.2-SNAPSHOT</version>
+ <version>3.0.0-SNAPSHOT</version>
<packaging>pom</packaging>
<name>Apache Curator</name>
[5/8] curator git commit: [CURATOR-160] Made Configurable optional.
Fix storingStatIn in GetConfigBuilderImpl.
Posted by ra...@apache.org.
[CURATOR-160] Made Configurable optional. Fix storingStatIn in GetConfigBuilderImpl.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/625441b5
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/625441b5
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/625441b5
Branch: refs/heads/CURATOR-160
Commit: 625441b588bf3a29bcec34edaa4d59a8fb720846
Parents: 59292d8
Author: Ioannis Canellos <io...@gmail.com>
Authored: Fri Apr 17 12:57:28 2015 +0300
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 13:00:30 2015 +0300
----------------------------------------------------------------------
.../api/BackgroundStatConfigEnsembleable.java | 24 ++++++++++
.../api/BackgroundStatConfigurable.java | 24 ----------
.../api/BackgroundStatEnsembleable.java | 24 ++++++++++
.../curator/framework/api/Configurable.java | 2 +-
.../JoinBackgroundStatConfigEnsembleable.java | 30 ++++++++++++
.../api/JoinBackgroundStatConfigurable.java | 30 ------------
.../LeaveBackgroundStatConfigEnsembleable.java | 30 ++++++++++++
.../api/LeaveBackgroundStatConfigurable.java | 30 ------------
.../curator/framework/api/ReconfigBuilder.java | 6 +--
.../framework/imps/GetConfigBuilderImpl.java | 2 +-
.../framework/imps/ReconfigBuilderImpl.java | 48 ++++++++++----------
.../framework/imps/TestReconfiguration.java | 32 ++++++-------
12 files changed, 153 insertions(+), 129 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigEnsembleable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigEnsembleable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigEnsembleable.java
new file mode 100644
index 0000000..f109b0f
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigEnsembleable.java
@@ -0,0 +1,24 @@
+/**
+ * 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 BackgroundStatConfigEnsembleable<T> extends
+ Configurable<BackgroundStatEnsembleable<T>>,
+ BackgroundStatEnsembleable<T> {
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java
deleted file mode 100644
index e46ba89..0000000
--- a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java
+++ /dev/null
@@ -1,24 +0,0 @@
-/**
- * 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 BackgroundStatConfigurable<T> extends
- BackgroundStatable<Configurable<T>>,
- Configurable<T> {
-}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatEnsembleable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatEnsembleable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatEnsembleable.java
new file mode 100644
index 0000000..62f666d
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatEnsembleable.java
@@ -0,0 +1,24 @@
+/**
+ * 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 BackgroundStatEnsembleable<T> extends
+ BackgroundStatable<Ensembleable<T>>,
+ Ensembleable<T> {
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
index a47f9d0..39e2c0c 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
@@ -25,5 +25,5 @@ public interface Configurable<T> {
* @param config The version of the configuration.
* @throws Exception
*/
- Ensembleable<T> fromConfig(long config) throws Exception;
+ BackgroundStatEnsembleable<byte[]> fromConfig(long config) throws Exception;
}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigEnsembleable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigEnsembleable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigEnsembleable.java
new file mode 100644
index 0000000..ba45efd
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigEnsembleable.java
@@ -0,0 +1,30 @@
+/**
+ * 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;
+
+/**
+ * An incremental reconfiguration builder.
+ * This builder has access only to the incremental reconfiguration methods joining and leaving, so that we prevent
+ * mixing concepts that can't be used together.
+ */
+public interface JoinBackgroundStatConfigEnsembleable extends
+ Joinable<BackgroundStatConfigEnsembleable<byte[]>>,
+ BackgroundStatConfigEnsembleable<byte[]> {
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java
deleted file mode 100644
index fb18c0c..0000000
--- a/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java
+++ /dev/null
@@ -1,30 +0,0 @@
-/**
- * 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;
-
-/**
- * An incremental reconfiguration builder.
- * This builder has access only to the incremental reconfiguration methods joining and leaving, so that we prevent
- * mixing concepts that can't be used together.
- */
-public interface JoinBackgroundStatConfigurable extends
- Joinable<BackgroundStatConfigurable<byte[]>>,
- BackgroundStatConfigurable<byte[]> {
-
-}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigEnsembleable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigEnsembleable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigEnsembleable.java
new file mode 100644
index 0000000..a6316c1
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigEnsembleable.java
@@ -0,0 +1,30 @@
+/**
+ * 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;
+
+/**
+ * An non-incremental reconfiguration builder.
+ * This builder has access only to the non-incremental reconfiguration methods withMembers, so that we prevent
+ * mixing concepts that can't be used together.
+ */
+public interface LeaveBackgroundStatConfigEnsembleable extends
+ Leaveable<BackgroundStatConfigEnsembleable<byte[]>>,
+ BackgroundStatConfigEnsembleable<byte[]> {
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java
deleted file mode 100644
index 196ffca..0000000
--- a/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java
+++ /dev/null
@@ -1,30 +0,0 @@
-/**
- * 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;
-
-/**
- * An non-incremental reconfiguration builder.
- * This builder has access only to the non-incremental reconfiguration methods withMembers, so that we prevent
- * mixing concepts that can't be used together.
- */
-public interface LeaveBackgroundStatConfigurable extends
- Leaveable<BackgroundStatConfigurable<byte[]>>,
- BackgroundStatConfigurable<byte[]> {
-
-}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java b/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
index 96ebdf7..f05b99e 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
@@ -19,8 +19,8 @@
package org.apache.curator.framework.api;
public interface ReconfigBuilder extends
- Joinable<LeaveBackgroundStatConfigurable>,
- Leaveable<JoinBackgroundStatConfigurable>,
- Memberable<BackgroundStatConfigurable<byte[]>> {
+ Joinable<LeaveBackgroundStatConfigEnsembleable>,
+ Leaveable<JoinBackgroundStatConfigEnsembleable>,
+ Memberable<BackgroundStatConfigEnsembleable<byte[]>> {
}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
index 59a621a..d331a0a 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
@@ -50,7 +50,7 @@ public class GetConfigBuilderImpl implements GetConfigBuilder, BackgroundOperati
@Override
public Ensembleable<byte[]> storingStatIn(Stat stat) {
- this.stat = new Stat();
+ this.stat = stat;
return this;
}
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
index 7a33297..5489691 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
@@ -21,13 +21,13 @@ package org.apache.curator.framework.imps;
import org.apache.curator.RetryLoop;
import org.apache.curator.TimeTrace;
import org.apache.curator.framework.api.BackgroundCallback;
-import org.apache.curator.framework.api.BackgroundStatConfigurable;
-import org.apache.curator.framework.api.Configurable;
+import org.apache.curator.framework.api.BackgroundStatConfigEnsembleable;
+import org.apache.curator.framework.api.BackgroundStatEnsembleable;
import org.apache.curator.framework.api.CuratorEvent;
import org.apache.curator.framework.api.CuratorEventType;
import org.apache.curator.framework.api.Ensembleable;
-import org.apache.curator.framework.api.JoinBackgroundStatConfigurable;
-import org.apache.curator.framework.api.LeaveBackgroundStatConfigurable;
+import org.apache.curator.framework.api.JoinBackgroundStatConfigEnsembleable;
+import org.apache.curator.framework.api.LeaveBackgroundStatConfigEnsembleable;
import org.apache.curator.framework.api.ReconfigBuilder;
import org.apache.zookeeper.AsyncCallback;
import org.apache.zookeeper.data.Stat;
@@ -46,7 +46,7 @@ public class ReconfigBuilderImpl implements ReconfigBuilder {
this.client = client;
}
- private static class ReconfigBuilderBase implements BackgroundStatConfigurable<byte[]>, Ensembleable<byte[]>, BackgroundOperation<EnsembleServersAndConfig> {
+ private static class ReconfigBuilderBase implements BackgroundStatConfigEnsembleable<byte[]>, Ensembleable<byte[]>, BackgroundOperation<EnsembleServersAndConfig> {
final CuratorFrameworkImpl client;
final List<String> joiningServers = new LinkedList<String>();
@@ -62,49 +62,49 @@ public class ReconfigBuilderImpl implements ReconfigBuilder {
}
@Override
- public Configurable<byte[]> inBackground() {
+ public Ensembleable<byte[]> inBackground() {
backgrounding = new Backgrounding();
return this;
}
@Override
- public Configurable<byte[]> inBackground(Object context) {
+ public Ensembleable<byte[]> inBackground(Object context) {
backgrounding = new Backgrounding(context);
return this;
}
@Override
- public Configurable<byte[]> inBackground(BackgroundCallback callback) {
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback) {
backgrounding = new Backgrounding(callback);
return this;
}
@Override
- public Configurable<byte[]> inBackground(BackgroundCallback callback, Object context) {
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback, Object context) {
backgrounding = new Backgrounding(callback, context);
return this;
}
@Override
- public Configurable<byte[]> inBackground(BackgroundCallback callback, Executor executor) {
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback, Executor executor) {
backgrounding = new Backgrounding(callback, executor);
return this;
}
@Override
- public Configurable<byte[]> inBackground(BackgroundCallback callback, Object context, Executor executor) {
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback, Object context, Executor executor) {
backgrounding = new Backgrounding(client, callback, context, executor);
return this;
}
@Override
- public Ensembleable<byte[]> fromConfig(long config) throws Exception {
+ public BackgroundStatEnsembleable<byte[]> fromConfig(long config) throws Exception {
this.config = config;
return this;
}
@Override
- public Configurable<byte[]> storingStatIn(Stat stat) {
+ public Ensembleable<byte[]> storingStatIn(Stat stat) {
this.stat = stat;
return this;
}
@@ -166,27 +166,27 @@ public class ReconfigBuilderImpl implements ReconfigBuilder {
}
}
- private static class JoinReconfigBuilder extends ReconfigBuilderBase implements JoinBackgroundStatConfigurable {
+ private static class JoinReconfigBuilderConfig extends ReconfigBuilderBase implements JoinBackgroundStatConfigEnsembleable {
- private JoinReconfigBuilder(CuratorFrameworkImpl client) {
+ private JoinReconfigBuilderConfig(CuratorFrameworkImpl client) {
super(client);
}
@Override
- public BackgroundStatConfigurable<byte[]> joining(String... servers) {
+ public BackgroundStatConfigEnsembleable<byte[]> joining(String... servers) {
joiningServers.addAll(Arrays.asList(servers));
return this;
}
}
- private static class LeaveReconfigBuilder extends ReconfigBuilderBase implements LeaveBackgroundStatConfigurable {
+ private static class LeaveReconfigBuilderConfig extends ReconfigBuilderBase implements LeaveBackgroundStatConfigEnsembleable {
- private LeaveReconfigBuilder(CuratorFrameworkImpl client) {
+ private LeaveReconfigBuilderConfig(CuratorFrameworkImpl client) {
super(client);
}
@Override
- public BackgroundStatConfigurable<byte[]> leaving(String... servers) {
+ public BackgroundStatConfigEnsembleable<byte[]> leaving(String... servers) {
leavingServers.addAll(Arrays.asList(servers));
return this;
}
@@ -194,21 +194,21 @@ public class ReconfigBuilderImpl implements ReconfigBuilder {
@Override
- public LeaveBackgroundStatConfigurable joining(String... servers) {
- LeaveReconfigBuilder builder = new LeaveReconfigBuilder(client);
+ public LeaveBackgroundStatConfigEnsembleable joining(String... servers) {
+ LeaveReconfigBuilderConfig builder = new LeaveReconfigBuilderConfig(client);
builder.joiningServers.addAll(Arrays.asList(servers));
return builder;
}
@Override
- public JoinBackgroundStatConfigurable leaving(String... servers) {
- JoinReconfigBuilder builder = new JoinReconfigBuilder(client);
+ public JoinBackgroundStatConfigEnsembleable leaving(String... servers) {
+ JoinReconfigBuilderConfig builder = new JoinReconfigBuilderConfig(client);
builder.leavingServers.addAll(Arrays.asList(servers));
return builder;
}
@Override
- public BackgroundStatConfigurable<byte[]> withMembers(String... servers) {
+ public BackgroundStatConfigEnsembleable<byte[]> withMembers(String... servers) {
ReconfigBuilderBase builder = new ReconfigBuilderBase(client);
builder.members.addAll(Arrays.asList(servers));
return builder;
http://git-wip-us.apache.org/repos/asf/curator/blob/625441b5/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
index faec551..44f9d00 100644
--- a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
+++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
@@ -101,14 +101,14 @@ public class TestReconfiguration {
String server2 = getServerString(qv, cluster, 2L);
//Remove Servers
- bytes = client.reconfig().leaving("1").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ bytes = client.reconfig().leaving("1").fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 4);
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
- bytes = client.reconfig().leaving("2").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ bytes = client.reconfig().leaving("2").fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 3);
@@ -116,14 +116,14 @@ public class TestReconfiguration {
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
//Add Servers
- bytes = client.reconfig().joining("server.2=" + server2).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ bytes = client.reconfig().joining("server.2=" + server2).fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 4);
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
- bytes = client.reconfig().joining("server.1=" + server1).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ bytes = client.reconfig().joining("server.1=" + server1).fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 5);
@@ -157,27 +157,27 @@ public class TestReconfiguration {
//Remove Servers
- client.reconfig().leaving("1").inBackground(callback).fromConfig(qv.getVersion()).forEnsemble();
+ client.reconfig().leaving("1").fromConfig(qv.getVersion()).inBackground(callback).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
qv = getQuorumVerifier(bytes.get());
Assert.assertEquals(qv.getAllMembers().size(), 4);
- client.reconfig().leaving("2").inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ client.reconfig().leaving("2").fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
qv = getQuorumVerifier(bytes.get());
Assert.assertEquals(qv.getAllMembers().size(), 3);
//Add Servers
- client.reconfig().joining("server.2=" + server2).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ client.reconfig().joining("server.2=" + server2).fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
qv = getQuorumVerifier(bytes.get());
Assert.assertEquals(qv.getAllMembers().size(), 4);
- client.reconfig().joining("server.1=" + server1).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ client.reconfig().joining("server.1=" + server1).fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString1to5);
qv = getQuorumVerifier(bytes.get());
@@ -203,7 +203,7 @@ public class TestReconfiguration {
"server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 4);
@@ -214,7 +214,7 @@ public class TestReconfiguration {
.withMembers("server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 3);
@@ -228,7 +228,7 @@ public class TestReconfiguration {
"server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 4);
@@ -241,7 +241,7 @@ public class TestReconfiguration {
"server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).storingStatIn(stat).forEnsemble();
qv = getQuorumVerifier(bytes);
Assert.assertEquals(qv.getAllMembers().size(), 5);
@@ -279,7 +279,7 @@ public class TestReconfiguration {
"server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
qv = getQuorumVerifier(bytes.get());
@@ -289,7 +289,7 @@ public class TestReconfiguration {
.withMembers("server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString3to5);
qv = getQuorumVerifier(bytes.get());
@@ -301,7 +301,7 @@ public class TestReconfiguration {
"server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString2to5);
qv = getQuorumVerifier(bytes.get());
@@ -313,7 +313,7 @@ public class TestReconfiguration {
"server.3=" + server3,
"server.4=" + server4,
"server.5=" + server5)
- .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ .fromConfig(qv.getVersion()).inBackground(callback, latch).forEnsemble();
waitOnDelegateListener.waitForEvent();
Assert.assertEquals(dynamicEnsembleProvider.getConnectionString(), connectionString1to5);
qv = getQuorumVerifier(bytes.get());
[4/8] curator git commit: [CURATOR-160] Add builders and dsl for
ZooKeeper's config and reconfig methods.
Posted by ra...@apache.org.
[CURATOR-160] Add builders and dsl for ZooKeeper's config and reconfig methods.
Project: http://git-wip-us.apache.org/repos/asf/curator/repo
Commit: http://git-wip-us.apache.org/repos/asf/curator/commit/0e92b562
Tree: http://git-wip-us.apache.org/repos/asf/curator/tree/0e92b562
Diff: http://git-wip-us.apache.org/repos/asf/curator/diff/0e92b562
Branch: refs/heads/CURATOR-160
Commit: 0e92b562dfb3ac0ec6bfae82b5690473f5ca7594
Parents: e11ef78
Author: Ioannis Canellos <io...@gmail.com>
Authored: Thu Nov 6 17:34:47 2014 +0200
Committer: Ioannis Canellos <io...@gmail.com>
Committed: Fri Apr 17 13:00:30 2015 +0300
----------------------------------------------------------------------
.../main/java/org/apache/curator/RetryLoop.java | 3 +-
.../curator/framework/CuratorFramework.java | 12 +
.../api/BackgroundStatConfigurable.java | 24 ++
.../framework/api/BackgroundStatable.java | 24 ++
.../curator/framework/api/Configurable.java | 29 ++
.../curator/framework/api/CuratorEventType.java | 10 +
.../curator/framework/api/Ensembleable.java | 24 ++
.../curator/framework/api/GetConfigBuilder.java | 27 ++
.../api/JoinBackgroundStatConfigurable.java | 30 ++
.../apache/curator/framework/api/Joinable.java | 31 ++
.../api/LeaveBackgroundStatConfigurable.java | 30 ++
.../apache/curator/framework/api/Leaveable.java | 30 ++
.../curator/framework/api/Memberable.java | 31 ++
.../curator/framework/api/ReconfigBuilder.java | 26 ++
.../framework/imps/CuratorFrameworkImpl.java | 10 +
.../imps/EnsembleServersAndConfig.java | 52 ++++
.../framework/imps/GetConfigBuilderImpl.java | 187 ++++++++++++
.../framework/imps/ReconfigBuilderImpl.java | 216 +++++++++++++
.../framework/imps/TestReconfiguration.java | 305 +++++++++++++++++++
.../org/apache/curator/test/InstanceSpec.java | 4 +
.../org/apache/curator/test/TestingCluster.java | 1 +
21 files changed, 1105 insertions(+), 1 deletion(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/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 6b66e82..065ebef 100644
--- a/curator-client/src/main/java/org/apache/curator/RetryLoop.java
+++ b/curator-client/src/main/java/org/apache/curator/RetryLoop.java
@@ -150,7 +150,8 @@ public class RetryLoop
return (rc == KeeperException.Code.CONNECTIONLOSS.intValue()) ||
(rc == KeeperException.Code.OPERATIONTIMEOUT.intValue()) ||
(rc == KeeperException.Code.SESSIONMOVED.intValue()) ||
- (rc == KeeperException.Code.SESSIONEXPIRED.intValue());
+ (rc == KeeperException.Code.SESSIONEXPIRED.intValue()) ||
+ (rc == KeeperException.Code.NEWCONFIGNOQUORUM.intValue());
}
/**
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/CuratorFramework.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFramework.java b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFramework.java
index 9c23ddb..181e4e8 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/CuratorFramework.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/CuratorFramework.java
@@ -122,6 +122,18 @@ public interface CuratorFramework extends Closeable
public SetACLBuilder setACL();
/**
+ * Start a reconfig builder
+ * @return builder object
+ */
+ public ReconfigBuilder reconfig();
+
+ /**
+ * Start a getConfig builder
+ * @return
+ */
+ public GetConfigBuilder getConfig();
+
+ /**
* Start a transaction builder
*
* @return builder object
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java
new file mode 100644
index 0000000..e46ba89
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatConfigurable.java
@@ -0,0 +1,24 @@
+/**
+ * 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 BackgroundStatConfigurable<T> extends
+ BackgroundStatable<Configurable<T>>,
+ Configurable<T> {
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatable.java
new file mode 100644
index 0000000..77c4e96
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/BackgroundStatable.java
@@ -0,0 +1,24 @@
+/**
+ * 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 BackgroundStatable<T> extends
+ Backgroundable<T>,
+ Statable<T> {
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
new file mode 100644
index 0000000..a47f9d0
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/Configurable.java
@@ -0,0 +1,29 @@
+/**
+ * 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 Configurable<T> {
+
+ /**
+ * Sets the configuration version to use.
+ * @param config The version of the configuration.
+ * @throws Exception
+ */
+ Ensembleable<T> fromConfig(long config) throws Exception;
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/CuratorEventType.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/CuratorEventType.java b/curator-framework/src/main/java/org/apache/curator/framework/api/CuratorEventType.java
index 684d11b..50e9195 100644
--- a/curator-framework/src/main/java/org/apache/curator/framework/api/CuratorEventType.java
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/CuratorEventType.java
@@ -69,6 +69,16 @@ public enum CuratorEventType
SET_ACL,
/**
+ * Corresponds to {@link CuratorFramework#getConfig()}
+ */
+ GET_CONFIG,
+
+ /**
+ * Corresponds to {@link CuratorFramework#reconfig()}
+ */
+ RECONFIG,
+
+ /**
* Corresponds to {@link Watchable#usingWatcher(Watcher)} or {@link Watchable#watched()}
*/
WATCHED,
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/Ensembleable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/Ensembleable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/Ensembleable.java
new file mode 100644
index 0000000..c8a82fe
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/Ensembleable.java
@@ -0,0 +1,24 @@
+/**
+ * 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 Ensembleable<T> {
+
+ T forEnsemble() throws Exception;
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/GetConfigBuilder.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/GetConfigBuilder.java b/curator-framework/src/main/java/org/apache/curator/framework/api/GetConfigBuilder.java
new file mode 100644
index 0000000..c2fdf6c
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/GetConfigBuilder.java
@@ -0,0 +1,27 @@
+/**
+ * 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 GetConfigBuilder extends
+ Watchable<BackgroundStatable<Ensembleable<byte[]>>>,
+ BackgroundStatable<Ensembleable<byte[]>>,
+ Ensembleable<byte[]> {
+}
+
+
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java
new file mode 100644
index 0000000..fb18c0c
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/JoinBackgroundStatConfigurable.java
@@ -0,0 +1,30 @@
+/**
+ * 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;
+
+/**
+ * An incremental reconfiguration builder.
+ * This builder has access only to the incremental reconfiguration methods joining and leaving, so that we prevent
+ * mixing concepts that can't be used together.
+ */
+public interface JoinBackgroundStatConfigurable extends
+ Joinable<BackgroundStatConfigurable<byte[]>>,
+ BackgroundStatConfigurable<byte[]> {
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/Joinable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/Joinable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/Joinable.java
new file mode 100644
index 0000000..dde5b1c
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/Joinable.java
@@ -0,0 +1,31 @@
+/**
+ * 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 Joinable<T> {
+
+ /**
+ * Adds one or more servers to joining the ensemble.
+ * The expected format is server.[id]=[hostname]:[peer port]:[election port]:[type];[client port]
+ * @param server The server to joining.
+ * @return this.
+ */
+ T joining(String... server);
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java
new file mode 100644
index 0000000..196ffca
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/LeaveBackgroundStatConfigurable.java
@@ -0,0 +1,30 @@
+/**
+ * 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;
+
+/**
+ * An non-incremental reconfiguration builder.
+ * This builder has access only to the non-incremental reconfiguration methods withMembers, so that we prevent
+ * mixing concepts that can't be used together.
+ */
+public interface LeaveBackgroundStatConfigurable extends
+ Leaveable<BackgroundStatConfigurable<byte[]>>,
+ BackgroundStatConfigurable<byte[]> {
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/Leaveable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/Leaveable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/Leaveable.java
new file mode 100644
index 0000000..a3c3358
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/Leaveable.java
@@ -0,0 +1,30 @@
+/**
+ * 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 Leaveable<T> {
+
+ /**
+ * Sets one or more servers to leaving the ensemble.
+ * @param server The server ids.
+ * @return this
+ */
+ T leaving(String... server);
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/Memberable.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/Memberable.java b/curator-framework/src/main/java/org/apache/curator/framework/api/Memberable.java
new file mode 100644
index 0000000..6ef54c1
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/Memberable.java
@@ -0,0 +1,31 @@
+/**
+ * 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 Memberable<T> {
+
+ /**
+ * Sets one or more members that are meant to be part of the ensemble.
+ * The expected format is server.[id]=[hostname]:[peer port]:[election port]:[type];[client port]
+ * @param server The server to add as a member of the ensemble.
+ * @return this.
+ */
+ T withMembers(String... server);
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java b/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
new file mode 100644
index 0000000..96ebdf7
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/api/ReconfigBuilder.java
@@ -0,0 +1,26 @@
+/**
+ * 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 ReconfigBuilder extends
+ Joinable<LeaveBackgroundStatConfigurable>,
+ Leaveable<JoinBackgroundStatConfigurable>,
+ Memberable<BackgroundStatConfigurable<byte[]>> {
+
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/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 5034ed9..b9b9c31 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
@@ -398,6 +398,16 @@ public class CuratorFrameworkImpl implements CuratorFramework
}
@Override
+ public ReconfigBuilder reconfig() {
+ return new ReconfigBuilderImpl(this);
+ }
+
+ @Override
+ public GetConfigBuilder getConfig() {
+ return new GetConfigBuilderImpl(this);
+ }
+
+ @Override
public CuratorTransaction inTransaction()
{
Preconditions.checkState(getState() == CuratorFrameworkState.STARTED, "instance must be started before calling this method");
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleServersAndConfig.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleServersAndConfig.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleServersAndConfig.java
new file mode 100644
index 0000000..df78aa7
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/EnsembleServersAndConfig.java
@@ -0,0 +1,52 @@
+/**
+ * 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 java.util.Collections;
+import java.util.List;
+
+class EnsembleServersAndConfig {
+ private final List<String> joiningServers;
+ private final List<String> leavingServers;
+ private final List<String> members;
+ private final long config;
+
+ EnsembleServersAndConfig(List<String> joiningServers, List<String> leavingServers, List<String> members, long config) {
+ this.joiningServers = joiningServers.isEmpty() ? null : Collections.unmodifiableList(joiningServers);
+ this.leavingServers = leavingServers.isEmpty() ? null : Collections.unmodifiableList(leavingServers);
+ this.members = members.isEmpty() ? null : Collections.unmodifiableList(members);
+ this.config = config;
+ }
+
+ public List<String> getJoiningServers() {
+ return joiningServers;
+ }
+
+ public List<String> getLeavingServers() {
+ return leavingServers;
+ }
+
+ public List<String> getMembers() {
+ return members;
+ }
+
+ public long getConfig() {
+ return config;
+ }
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
new file mode 100644
index 0000000..54b1862
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/GetConfigBuilderImpl.java
@@ -0,0 +1,187 @@
+/**
+ * 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 org.apache.curator.RetryLoop;
+import org.apache.curator.TimeTrace;
+import org.apache.curator.framework.api.BackgroundCallback;
+import org.apache.curator.framework.api.BackgroundStatable;
+import org.apache.curator.framework.api.CuratorEvent;
+import org.apache.curator.framework.api.CuratorEventType;
+import org.apache.curator.framework.api.CuratorWatcher;
+import org.apache.curator.framework.api.Ensembleable;
+import org.apache.curator.framework.api.GetConfigBuilder;
+import org.apache.zookeeper.AsyncCallback;
+import org.apache.zookeeper.Watcher;
+import org.apache.zookeeper.data.Stat;
+
+import java.util.concurrent.Callable;
+import java.util.concurrent.Executor;
+
+public class GetConfigBuilderImpl implements GetConfigBuilder, BackgroundOperation<Void> {
+
+ private final CuratorFrameworkImpl client;
+
+ private Backgrounding backgrounding;
+ private Watching watching;
+ private Stat stat;
+
+ public GetConfigBuilderImpl(CuratorFrameworkImpl client) {
+ this.client = client;
+ backgrounding = new Backgrounding();
+ watching = new Watching();
+ }
+
+ @Override
+ public Ensembleable<byte[]> storingStatIn(Stat stat) {
+ this.stat = new Stat();
+ return this;
+ }
+
+ @Override
+ public BackgroundStatable<Ensembleable<byte[]>> watched() {
+ watching = new Watching(true);
+ return this;
+ }
+
+ @Override
+ public GetConfigBuilder usingWatcher(Watcher watcher) {
+ watching = new Watching(client, watcher);
+ return this;
+ }
+
+ @Override
+ public GetConfigBuilder usingWatcher(final CuratorWatcher watcher) {
+ watching = new Watching(client, watcher);
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> inBackground() {
+ backgrounding = new Backgrounding();
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> inBackground(Object context) {
+ backgrounding = new Backgrounding(context);
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback) {
+ backgrounding = new Backgrounding(callback);
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback, Object context) {
+ backgrounding = new Backgrounding(callback, context);
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback, Executor executor) {
+ backgrounding = new Backgrounding(callback, executor);
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> inBackground(BackgroundCallback callback, Object context, Executor executor) {
+ backgrounding = new Backgrounding(client, callback, context, executor);
+ return this;
+ }
+
+ private void performBackgroundOperation() {
+ try {
+ client.getZooKeeper().getConfig(watching.getWatcher(),
+ new AsyncCallback.DataCallback() {
+ @Override
+ public void processResult(int rc, String path, Object ctx, byte[] data, Stat stat) {
+ try {
+ CuratorEvent event = new CuratorEventImpl(client, CuratorEventType.GET_CONFIG,
+ rc, path, null, ctx, stat, data, null, null, null);
+ backgrounding.getCallback().processResult(client, event);
+ } catch (Exception e) {
+ throw new RuntimeException(e);
+ }
+ }
+ }, backgrounding.getContext());
+ } catch (Exception e) {
+ throw new RuntimeException(e);
+ }
+ }
+
+ @Override
+ public byte[] forEnsemble() throws Exception {
+ byte[] responseData = null;
+ if ( backgrounding.inBackground() )
+ {
+ client.processBackgroundOperation(new OperationAndData<Void>(this, null, backgrounding.getCallback(), null, backgrounding.getContext()), null);
+ }
+ else
+ {
+ responseData = configInForeground();
+ }
+ return responseData;
+ }
+
+ @Override
+ public void performBackgroundOperation(final OperationAndData<Void> operationAndData) throws Exception
+ {
+ final TimeTrace trace = client.getZookeeperClient().startTracer("GetDataBuilderImpl-Background");
+ AsyncCallback.DataCallback callback = new AsyncCallback.DataCallback()
+ {
+ @Override
+ public void processResult(int rc, String path, Object ctx, byte[] data, Stat stat)
+ {
+ trace.commit();
+ CuratorEvent event = new CuratorEventImpl(client, CuratorEventType.GET_CONFIG, rc, path, null, ctx, stat, data, null, null, null);
+ client.processBackgroundOperation(operationAndData, event);
+ }
+ };
+ if ( watching.isWatched() )
+ {
+ client.getZooKeeper().getConfig(watching.getWatcher(), callback, backgrounding.getContext());
+ }
+ else
+ {
+ client.getZooKeeper().getConfig(false, callback, backgrounding.getContext());
+ }
+ }
+
+
+ private byte[] configInForeground() throws Exception {
+ TimeTrace trace = client.getZookeeperClient().startTracer("GetConfigBuilderImpl-Foreground");
+ try {
+ return RetryLoop.callWithRetry
+ (
+ client.getZookeeperClient(),
+ new Callable<byte[]>() {
+ @Override
+ public byte[] call() throws Exception {
+ return client.getZooKeeper().getConfig(watching.getWatcher(), stat);
+ }
+ }
+ );
+ } finally {
+ trace.commit();
+ }
+ }
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java b/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
new file mode 100644
index 0000000..7a33297
--- /dev/null
+++ b/curator-framework/src/main/java/org/apache/curator/framework/imps/ReconfigBuilderImpl.java
@@ -0,0 +1,216 @@
+/**
+ * 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 org.apache.curator.RetryLoop;
+import org.apache.curator.TimeTrace;
+import org.apache.curator.framework.api.BackgroundCallback;
+import org.apache.curator.framework.api.BackgroundStatConfigurable;
+import org.apache.curator.framework.api.Configurable;
+import org.apache.curator.framework.api.CuratorEvent;
+import org.apache.curator.framework.api.CuratorEventType;
+import org.apache.curator.framework.api.Ensembleable;
+import org.apache.curator.framework.api.JoinBackgroundStatConfigurable;
+import org.apache.curator.framework.api.LeaveBackgroundStatConfigurable;
+import org.apache.curator.framework.api.ReconfigBuilder;
+import org.apache.zookeeper.AsyncCallback;
+import org.apache.zookeeper.data.Stat;
+
+import java.util.Arrays;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.concurrent.Callable;
+import java.util.concurrent.Executor;
+
+public class ReconfigBuilderImpl implements ReconfigBuilder {
+
+ private final CuratorFrameworkImpl client;
+
+ public ReconfigBuilderImpl(CuratorFrameworkImpl client) {
+ this.client = client;
+ }
+
+ private static class ReconfigBuilderBase implements BackgroundStatConfigurable<byte[]>, Ensembleable<byte[]>, BackgroundOperation<EnsembleServersAndConfig> {
+
+ final CuratorFrameworkImpl client;
+ final List<String> joiningServers = new LinkedList<String>();
+ final List<String> leavingServers = new LinkedList<String>();
+ final List<String> members = new LinkedList<String>();
+ Backgrounding backgrounding;
+ Stat stat;
+ long config;
+
+ private ReconfigBuilderBase(CuratorFrameworkImpl client) {
+ this.client = client;
+ backgrounding = new Backgrounding();
+ }
+
+ @Override
+ public Configurable<byte[]> inBackground() {
+ backgrounding = new Backgrounding();
+ return this;
+ }
+
+ @Override
+ public Configurable<byte[]> inBackground(Object context) {
+ backgrounding = new Backgrounding(context);
+ return this;
+ }
+
+ @Override
+ public Configurable<byte[]> inBackground(BackgroundCallback callback) {
+ backgrounding = new Backgrounding(callback);
+ return this;
+ }
+
+ @Override
+ public Configurable<byte[]> inBackground(BackgroundCallback callback, Object context) {
+ backgrounding = new Backgrounding(callback, context);
+ return this;
+ }
+
+ @Override
+ public Configurable<byte[]> inBackground(BackgroundCallback callback, Executor executor) {
+ backgrounding = new Backgrounding(callback, executor);
+ return this;
+ }
+
+ @Override
+ public Configurable<byte[]> inBackground(BackgroundCallback callback, Object context, Executor executor) {
+ backgrounding = new Backgrounding(client, callback, context, executor);
+ return this;
+ }
+
+ @Override
+ public Ensembleable<byte[]> fromConfig(long config) throws Exception {
+ this.config = config;
+ return this;
+ }
+
+ @Override
+ public Configurable<byte[]> storingStatIn(Stat stat) {
+ this.stat = stat;
+ return this;
+ }
+
+ @Override
+ public byte[] forEnsemble() throws Exception {
+ if (backgrounding.inBackground()) {
+ client.processBackgroundOperation(new OperationAndData<EnsembleServersAndConfig>(this,
+ new EnsembleServersAndConfig(joiningServers, leavingServers, members, config),
+ backgrounding.getCallback(), null, backgrounding.getContext()), null);
+ return new byte[0];
+ } else {
+ return ensembleInForeground();
+ }
+ }
+
+ private byte[] ensembleInForeground() throws Exception {
+ TimeTrace trace = client.getZookeeperClient().startTracer("ReconfigBuilderImpl-Foreground");
+ byte[] responseData = RetryLoop.callWithRetry
+ (
+ client.getZookeeperClient(),
+ new Callable<byte[]>() {
+ @Override
+ public byte[] call() throws Exception {
+ return client.getZooKeeper().reconfig(
+ joiningServers.isEmpty() ? null : joiningServers,
+ leavingServers.isEmpty() ? null : leavingServers,
+ members.isEmpty() ? null : members,
+ config,
+ stat
+ );
+ }
+ }
+ );
+ trace.commit();
+ return responseData;
+ }
+
+ @Override
+ public void performBackgroundOperation(final OperationAndData<EnsembleServersAndConfig> operationAndData) throws Exception {
+ final TimeTrace trace = client.getZookeeperClient().startTracer("ReconfigBuilderImpl-Background");
+ AsyncCallback.DataCallback callback = new AsyncCallback.DataCallback() {
+ @Override
+ public void processResult(int rc, String path, Object ctx, byte[] data, Stat stat) {
+ trace.commit();
+ CuratorEvent event = new CuratorEventImpl(client, CuratorEventType.RECONFIG, rc, path, null, ctx, stat, data, null, null, null);
+ client.processBackgroundOperation(operationAndData, event);
+ }
+ };
+ client.getZooKeeper().reconfig(
+ operationAndData.getData().getJoiningServers(),
+ operationAndData.getData().getLeavingServers(),
+ operationAndData.getData().getMembers(),
+ operationAndData.getData().getConfig(),
+ callback,
+ operationAndData.getContext()
+ );
+
+ }
+ }
+
+ private static class JoinReconfigBuilder extends ReconfigBuilderBase implements JoinBackgroundStatConfigurable {
+
+ private JoinReconfigBuilder(CuratorFrameworkImpl client) {
+ super(client);
+ }
+
+ @Override
+ public BackgroundStatConfigurable<byte[]> joining(String... servers) {
+ joiningServers.addAll(Arrays.asList(servers));
+ return this;
+ }
+ }
+
+ private static class LeaveReconfigBuilder extends ReconfigBuilderBase implements LeaveBackgroundStatConfigurable {
+
+ private LeaveReconfigBuilder(CuratorFrameworkImpl client) {
+ super(client);
+ }
+
+ @Override
+ public BackgroundStatConfigurable<byte[]> leaving(String... servers) {
+ leavingServers.addAll(Arrays.asList(servers));
+ return this;
+ }
+ }
+
+
+ @Override
+ public LeaveBackgroundStatConfigurable joining(String... servers) {
+ LeaveReconfigBuilder builder = new LeaveReconfigBuilder(client);
+ builder.joiningServers.addAll(Arrays.asList(servers));
+ return builder;
+ }
+
+ @Override
+ public JoinBackgroundStatConfigurable leaving(String... servers) {
+ JoinReconfigBuilder builder = new JoinReconfigBuilder(client);
+ builder.leavingServers.addAll(Arrays.asList(servers));
+ return builder;
+ }
+
+ @Override
+ public BackgroundStatConfigurable<byte[]> withMembers(String... servers) {
+ ReconfigBuilderBase builder = new ReconfigBuilderBase(client);
+ builder.members.addAll(Arrays.asList(servers));
+ return builder;
+ }
+}
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
----------------------------------------------------------------------
diff --git a/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
new file mode 100644
index 0000000..e8896ae
--- /dev/null
+++ b/curator-framework/src/test/java/org/apache/curator/framework/imps/TestReconfiguration.java
@@ -0,0 +1,305 @@
+/**
+ * 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 org.apache.curator.framework.CuratorFramework;
+import org.apache.curator.framework.CuratorFrameworkFactory;
+import org.apache.curator.framework.api.BackgroundCallback;
+import org.apache.curator.framework.api.CuratorEvent;
+import org.apache.curator.retry.RetryOneTime;
+import org.apache.curator.test.InstanceSpec;
+import org.apache.curator.test.TestingCluster;
+import org.apache.zookeeper.data.Stat;
+import org.apache.zookeeper.server.quorum.flexible.QuorumMaj;
+import org.apache.zookeeper.server.quorum.flexible.QuorumVerifier;
+import org.testng.Assert;
+import org.testng.annotations.AfterMethod;
+import org.testng.annotations.BeforeMethod;
+import org.testng.annotations.Test;
+
+import java.io.IOException;
+import java.io.StringReader;
+import java.util.Properties;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicReference;
+
+public class TestReconfiguration {
+
+ TestingCluster cluster;
+
+ @BeforeMethod
+ public void setup() throws Exception {
+ cluster = new TestingCluster(5);
+ cluster.start();
+ }
+
+ @AfterMethod
+ public void tearDown() throws IOException {
+ cluster.close();
+ }
+
+ @Test
+ public void testSyncIncremental() throws Exception {
+ CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
+ client.start();
+ client.blockUntilConnected();
+ try {
+ Stat stat = new Stat();
+ byte[] bytes = client.getConfig().storingStatIn(stat).forEnsemble();
+ Assert.assertNotNull(bytes);
+ QuorumVerifier qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+
+ //Remove Servers
+ bytes = client.reconfig().leaving("1").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ bytes = client.reconfig().leaving("2").storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ //Add Servers
+ bytes = client.reconfig().joining("server.1=" + server1).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ bytes = client.reconfig().joining("server.2=" + server2).storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ } finally {
+ client.close();
+ }
+ }
+
+ @Test
+ public void testAsyncIncremental() throws Exception {
+ CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
+ client.start();
+ client.blockUntilConnected();
+ try {
+ final AtomicReference<byte[]> bytes = new AtomicReference<byte[]>();
+ final BackgroundCallback callback = new BackgroundCallback() {
+ @Override
+ public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
+ bytes.set(event.getData());
+ ((CountDownLatch)event.getContext()).countDown();
+ }
+
+ };
+
+ CountDownLatch latch = new CountDownLatch(1);
+ client.getConfig().inBackground(callback, latch).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ Assert.assertNotNull(bytes.get());
+ QuorumVerifier qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+
+
+ //Remove Servers
+ latch = new CountDownLatch(1);
+ client.reconfig().leaving("1").inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ latch = new CountDownLatch(1);
+ client.reconfig().leaving("2").inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ //Add Servers
+ latch = new CountDownLatch(1);
+ client.reconfig().joining("server.1=" + server1).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ latch = new CountDownLatch(1);
+ client.reconfig().joining("server.2=" + server2).inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ } finally {
+ client.close();
+ }
+ }
+
+ @Test
+ public void testSyncNonIncremental() throws Exception {
+ CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
+ client.start();
+ client.blockUntilConnected();
+ try {
+ Stat stat = new Stat();
+ byte[] bytes = client.getConfig().storingStatIn(stat).forEnsemble();
+ Assert.assertNotNull(bytes);
+ QuorumVerifier qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+ String server3 = getServerString(qv, cluster, 3L);
+ String server4 = getServerString(qv, cluster, 4L);
+ String server5 = getServerString(qv, cluster, 5L);
+
+ //Remove Servers
+ bytes = client.reconfig()
+ .withMembers("server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ bytes = client.reconfig()
+ .withMembers("server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ //Add Servers
+ bytes = client.reconfig()
+ .withMembers("server.1=" + server1,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ bytes = client.reconfig()
+ .withMembers("server.1=" + server1,
+ "server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .storingStatIn(stat).fromConfig(qv.getVersion()).forEnsemble();
+ qv = getQuorumVerifier(bytes);
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ } finally {
+ client.close();
+ }
+ }
+
+ @Test
+ public void testAsyncNonIncremental() throws Exception {
+ CuratorFramework client = CuratorFrameworkFactory.newClient(cluster.getConnectString(), new RetryOneTime(1));
+ client.start();
+ client.blockUntilConnected();
+ try {
+ final AtomicReference<byte[]> bytes = new AtomicReference<byte[]>();
+ final BackgroundCallback callback = new BackgroundCallback() {
+ @Override
+ public void processResult(CuratorFramework client, CuratorEvent event) throws Exception {
+ bytes.set(event.getData());
+ ((CountDownLatch)event.getContext()).countDown();
+ }
+
+ };
+
+ CountDownLatch latch = new CountDownLatch(1);
+ client.getConfig().inBackground(callback, latch).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ Assert.assertNotNull(bytes.get());
+ QuorumVerifier qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ String server1 = getServerString(qv, cluster, 1L);
+ String server2 = getServerString(qv, cluster, 2L);
+ String server3 = getServerString(qv, cluster, 3L);
+ String server4 = getServerString(qv, cluster, 4L);
+ String server5 = getServerString(qv, cluster, 5L);
+
+ //Remove Servers
+ latch = new CountDownLatch(1);
+ client.reconfig()
+ .withMembers("server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ latch = new CountDownLatch(1);
+ client.reconfig()
+ .withMembers("server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 3);
+
+ //Add Servers
+ latch = new CountDownLatch(1);
+ client.reconfig()
+ .withMembers("server.1=" + server1,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 4);
+ latch = new CountDownLatch(1);
+ client.reconfig()
+ .withMembers("server.1=" + server1,
+ "server.2=" + server2,
+ "server.3=" + server3,
+ "server.4=" + server4,
+ "server.5=" + server5)
+ .inBackground(callback, latch).fromConfig(qv.getVersion()).forEnsemble();
+ latch.await(5, TimeUnit.SECONDS);
+ qv = getQuorumVerifier(bytes.get());
+ Assert.assertEquals(qv.getAllMembers().size(), 5);
+ } finally {
+ client.close();
+ }
+ }
+
+
+ static QuorumVerifier getQuorumVerifier(byte[] bytes) throws Exception {
+ Properties properties = new Properties();
+ properties.load(new StringReader(new String(bytes)));
+ return new QuorumMaj(properties);
+ }
+
+ static InstanceSpec getInstance(TestingCluster cluster, int id) {
+ for (InstanceSpec spec : cluster.getInstances()) {
+ if (spec.getServerId() == id) {
+ return spec;
+ }
+ }
+ throw new IllegalStateException("InstanceSpec with id:" + id + " not found");
+ }
+
+ static String getServerString(QuorumVerifier qv, TestingCluster cluster, long id) throws Exception {
+ String str = qv.getAllMembers().get(id).toString();
+ //check if connection string is already there.
+ if (str.contains(";")) {
+ return str;
+ } else {
+ return str + ";" + getInstance(cluster, (int) id).getConnectString();
+ }
+ }
+}
\ No newline at end of file
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-test/src/main/java/org/apache/curator/test/InstanceSpec.java
----------------------------------------------------------------------
diff --git a/curator-test/src/main/java/org/apache/curator/test/InstanceSpec.java b/curator-test/src/main/java/org/apache/curator/test/InstanceSpec.java
index b39a949..6d495df 100644
--- a/curator-test/src/main/java/org/apache/curator/test/InstanceSpec.java
+++ b/curator-test/src/main/java/org/apache/curator/test/InstanceSpec.java
@@ -70,6 +70,10 @@ public class InstanceSpec
private final int tickTime;
private final int maxClientCnxns;
+ public static void reset() {
+ nextServerId.set(1);
+ }
+
public static InstanceSpec newInstanceSpec()
{
return new InstanceSpec(null, -1, -1, -1, true, -1, -1, -1);
http://git-wip-us.apache.org/repos/asf/curator/blob/0e92b562/curator-test/src/main/java/org/apache/curator/test/TestingCluster.java
----------------------------------------------------------------------
diff --git a/curator-test/src/main/java/org/apache/curator/test/TestingCluster.java b/curator-test/src/main/java/org/apache/curator/test/TestingCluster.java
index cd86b72..f6bdbd8 100644
--- a/curator-test/src/main/java/org/apache/curator/test/TestingCluster.java
+++ b/curator-test/src/main/java/org/apache/curator/test/TestingCluster.java
@@ -249,6 +249,7 @@ public class TestingCluster implements Closeable
private static Map<InstanceSpec, Collection<InstanceSpec>> makeSpecs(int instanceQty)
{
+ InstanceSpec.reset();
ImmutableList.Builder<InstanceSpec> builder = ImmutableList.builder();
for ( int i = 0; i < instanceQty; ++i )
{