You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@zookeeper.apache.org by ph...@apache.org on 2018/04/24 23:43:05 UTC

zookeeper git commit: ZOOKEEPER-2415: SessionTest is using Thread deprecated API

Repository: zookeeper
Updated Branches:
  refs/heads/master cb6cae91c -> 3783ed586


ZOOKEEPER-2415: SessionTest is using Thread deprecated API

Refactored `SessionTest` class.

`testSessionTimeout()` has been extracted to a separate class and split into multiple test methods to validate the different behaviours in a more readable way.

Removed deprecated Thread API `suspend()` and reduced the server ticktime / session timeout for better performance.

Hopefully this will help on test's flakyness too.

Author: Andor Molnar <an...@cloudera.com>

Reviewers: phunt@apache.org

Closes #497 from anmolnar/ZOOKEEPER-2415 and squashes the following commits:

5f9f76c67 [Andor Molnar] ZOOKEEPER-2415. Reverted even more changes, because we don't need custom tickTime either anymore
8ed14e511 [Andor Molnar] ZOOKEEPER-2415. Moved session expiration test from ZooKeeperTestableTest class because it's redundant and more reliable
cfd18b64b [Andor Molnar] ZOOKEEPER-2415. Reset changes which are unrelated to this patch
76520025c [Andor Molnar] ZOOKEEPER-2415. Revert changes related to DisconnectableZooKeeper, because we don't need it for the new tests
6b5d26d0d [Andor Molnar] ZOOKEEPER-2415. Use ClientBase base class for the refactored unit tests
6026c591b [Andor Molnar] ZOOKEEPER-2415. Refactor testSessionTimeout() to live in separate class and not to use deprecated API. Also improved performance.

Change-Id: I33906f43de52a06e95ba3f19f8087033cd5857e2


Project: http://git-wip-us.apache.org/repos/asf/zookeeper/repo
Commit: http://git-wip-us.apache.org/repos/asf/zookeeper/commit/3783ed58
Tree: http://git-wip-us.apache.org/repos/asf/zookeeper/tree/3783ed58
Diff: http://git-wip-us.apache.org/repos/asf/zookeeper/diff/3783ed58

Branch: refs/heads/master
Commit: 3783ed58656f8719d7213329451f396f5c1e3dce
Parents: cb6cae9
Author: Andor Molnar <an...@cloudera.com>
Authored: Tue Apr 24 16:41:54 2018 -0700
Committer: Patrick Hunt <ph...@apache.org>
Committed: Tue Apr 24 16:41:54 2018 -0700

----------------------------------------------------------------------
 .../org/apache/zookeeper/TestableZooKeeper.java |   9 ++
 .../apache/zookeeper/ZooKeeperTestableTest.java |  59 ---------
 .../org/apache/zookeeper/test/SessionTest.java  |  67 ----------
 .../zookeeper/test/SessionTimeoutTest.java      | 129 +++++++++++++++++++
 4 files changed, 138 insertions(+), 126 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/zookeeper/blob/3783ed58/src/java/test/org/apache/zookeeper/TestableZooKeeper.java
----------------------------------------------------------------------
diff --git a/src/java/test/org/apache/zookeeper/TestableZooKeeper.java b/src/java/test/org/apache/zookeeper/TestableZooKeeper.java
index cd4c53c..bb1bd12 100644
--- a/src/java/test/org/apache/zookeeper/TestableZooKeeper.java
+++ b/src/java/test/org/apache/zookeeper/TestableZooKeeper.java
@@ -116,4 +116,13 @@ public class TestableZooKeeper extends ZooKeeperAdmin {
             Record response, WatchRegistration watchRegistration) throws InterruptedException {
         return cnxn.submitRequest(h, request, response, watchRegistration);
     }
+
+    /** Testing only!!! Really!!!! This is only here to test when the client
+     * disconnects from the server w/o sending a session disconnect (ie
+     * ending the session cleanly). The server will eventually notice the
+     * client is no longer pinging and will timeout the session.
+     */
+    public void disconnect() {
+        cnxn.disconnect();
+    }
 }

http://git-wip-us.apache.org/repos/asf/zookeeper/blob/3783ed58/src/java/test/org/apache/zookeeper/ZooKeeperTestableTest.java
----------------------------------------------------------------------
diff --git a/src/java/test/org/apache/zookeeper/ZooKeeperTestableTest.java b/src/java/test/org/apache/zookeeper/ZooKeeperTestableTest.java
deleted file mode 100644
index 01675df..0000000
--- a/src/java/test/org/apache/zookeeper/ZooKeeperTestableTest.java
+++ /dev/null
@@ -1,59 +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.zookeeper;
-
-import org.apache.zookeeper.test.ClientBase;
-import org.junit.Assert;
-import org.junit.Test;
-
-import java.io.IOException;
-import java.util.concurrent.CountDownLatch;
-import java.util.concurrent.TimeUnit;
-
-public class ZooKeeperTestableTest extends ClientBase {
-    @Test
-    public void testSessionExpiration() throws IOException, InterruptedException,
-            KeeperException {
-        ZooKeeper zk = createClient();
-
-        final CountDownLatch expirationLatch = new CountDownLatch(1);
-        Watcher watcher = new Watcher() {
-            @Override
-            public void process(WatchedEvent event) {
-                if ( event.getState() == Event.KeeperState.Expired ) {
-                    expirationLatch.countDown();
-                }
-            }
-        };
-        zk.exists("/foo", watcher);
-
-        zk.getTestable().injectSessionExpiration();
-        Assert.assertTrue(expirationLatch.await(5, TimeUnit.SECONDS));
-
-        boolean gotException = false;
-        try {
-            zk.exists("/foo", false);
-            Assert.fail("Should have thrown a SessionExpiredException");
-        } catch (KeeperException.SessionExpiredException e) {
-            // correct
-            gotException = true;
-        }
-        Assert.assertTrue(gotException);
-    }
-}

http://git-wip-us.apache.org/repos/asf/zookeeper/blob/3783ed58/src/java/test/org/apache/zookeeper/test/SessionTest.java
----------------------------------------------------------------------
diff --git a/src/java/test/org/apache/zookeeper/test/SessionTest.java b/src/java/test/org/apache/zookeeper/test/SessionTest.java
index 66b1be3..4a74a78 100644
--- a/src/java/test/org/apache/zookeeper/test/SessionTest.java
+++ b/src/java/test/org/apache/zookeeper/test/SessionTest.java
@@ -22,7 +22,6 @@ import static org.apache.zookeeper.test.ClientBase.CONNECTION_TIMEOUT;
 
 import java.io.File;
 import java.io.IOException;
-import java.util.ArrayList;
 import java.util.LinkedList;
 import java.util.List;
 import java.util.concurrent.CountDownLatch;
@@ -235,72 +234,6 @@ public class SessionTest extends ZKTestCase {
         Assert.assertEquals(KeeperException.Code.SESSIONEXPIRED.toString(), cb.toString());
     }
 
-    private List<Thread> findThreads(String name) {
-        int threadCount = Thread.activeCount();
-        Thread threads[] = new Thread[threadCount*2];
-        threadCount = Thread.enumerate(threads);
-        ArrayList<Thread> list = new ArrayList<Thread>();
-        for(int i = 0; i < threadCount; i++) {
-            if (threads[i].getName().indexOf(name) != -1) {
-                list.add(threads[i]);
-            }
-        }
-        return list;
-    }
-
-    /**
-     * Make sure ephemerals get cleaned up when a session times out.
-     */
-    @SuppressWarnings("deprecation")
-    @Test
-    public void testSessionTimeout() throws Exception {
-        final int TIMEOUT = 5000;
-        List<Thread> etBefore = findThreads("EventThread");
-        List<Thread> stBefore = findThreads("SendThread");
-        DisconnectableZooKeeper zk = createClient(TIMEOUT);
-        zk.create("/stest", new byte[0], Ids.OPEN_ACL_UNSAFE,
-                CreateMode.EPHEMERAL);
-
-        // Find the new event and send threads
-        List<Thread> etAfter = findThreads("EventThread");
-        List<Thread> stAfter = findThreads("SendThread");
-        Thread eventThread = null;
-        Thread sendThread = null;
-        for(Thread t: etAfter) {
-            if (!etBefore.contains(t)) {
-                eventThread = t;
-                break;
-            }
-        }
-        for(Thread t: stAfter) {
-            if (!stBefore.contains(t)) {
-                sendThread = t;
-                break;
-            }
-        }
-        sendThread.suspend();
-        //zk.disconnect();
-
-        Thread.sleep(TIMEOUT*2);
-        sendThread.resume();
-        eventThread.join(TIMEOUT);
-        Assert.assertFalse("EventThread is still running", eventThread.isAlive());
-
-        zk = createClient(TIMEOUT);
-        zk.create("/stest", new byte[0], Ids.OPEN_ACL_UNSAFE,
-                CreateMode.EPHEMERAL);
-        tearDown();
-        zk.close();
-        zk.disconnect();
-        setUp();
-
-        zk = createClient(TIMEOUT);
-        Assert.assertTrue(zk.exists("/stest", false) != null);
-        Thread.sleep(TIMEOUT*2);
-        Assert.assertTrue(zk.exists("/stest", false) == null);
-        zk.close();
-    }
-
     /**
      * Make sure that we cannot have two connections with the same
      * session id.

http://git-wip-us.apache.org/repos/asf/zookeeper/blob/3783ed58/src/java/test/org/apache/zookeeper/test/SessionTimeoutTest.java
----------------------------------------------------------------------
diff --git a/src/java/test/org/apache/zookeeper/test/SessionTimeoutTest.java b/src/java/test/org/apache/zookeeper/test/SessionTimeoutTest.java
new file mode 100644
index 0000000..09badae
--- /dev/null
+++ b/src/java/test/org/apache/zookeeper/test/SessionTimeoutTest.java
@@ -0,0 +1,129 @@
+/**
+ * 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.zookeeper.test;
+
+import org.apache.zookeeper.CreateMode;
+import org.apache.zookeeper.KeeperException;
+import org.apache.zookeeper.TestableZooKeeper;
+import org.apache.zookeeper.WatchedEvent;
+import org.apache.zookeeper.Watcher;
+import org.apache.zookeeper.ZooDefs;
+import org.apache.zookeeper.ZooKeeper;
+import org.apache.zookeeper.data.Stat;
+import org.junit.Assert;
+import org.junit.Before;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.File;
+import java.io.IOException;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.TimeUnit;
+
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
+
+public class SessionTimeoutTest extends ClientBase {
+    protected static final Logger LOG = LoggerFactory.getLogger(SessionTimeoutTest.class);
+
+    private TestableZooKeeper zk;
+
+    @Before
+    public void setUp() throws Exception {
+        super.setUp();
+        zk = createClient();
+    }
+
+    @Test
+    public void testSessionExpiration() throws InterruptedException,
+            KeeperException {
+        final CountDownLatch expirationLatch = new CountDownLatch(1);
+        Watcher watcher = new Watcher() {
+            @Override
+            public void process(WatchedEvent event) {
+                if ( event.getState() == Event.KeeperState.Expired ) {
+                    expirationLatch.countDown();
+                }
+            }
+        };
+        zk.exists("/foo", watcher);
+
+        zk.getTestable().injectSessionExpiration();
+        Assert.assertTrue(expirationLatch.await(5, TimeUnit.SECONDS));
+
+        boolean gotException = false;
+        try {
+            zk.exists("/foo", false);
+            Assert.fail("Should have thrown a SessionExpiredException");
+        } catch (KeeperException.SessionExpiredException e) {
+            // correct
+            gotException = true;
+        }
+        Assert.assertTrue(gotException);
+    }
+
+    /**
+     * Make sure ephemerals get cleaned up when session disconnects.
+     */
+    @Test
+    public void testSessionDisconnect() throws KeeperException, InterruptedException, IOException {
+        zk.create("/sdisconnect", new byte[0], ZooDefs.Ids.OPEN_ACL_UNSAFE,
+                CreateMode.EPHEMERAL);
+        assertNotNull("Ephemeral node has not been created", zk.exists("/sdisconnect", null));
+
+        zk.close();
+
+        zk = createClient();
+        assertNull("Ephemeral node shouldn't exist after client disconnect", zk.exists("/sdisconnect", null));
+    }
+
+    /**
+     * Make sure ephemerals are kept when session restores.
+     */
+    @Test
+    public void testSessionRestore() throws KeeperException, InterruptedException, IOException {
+        zk.create("/srestore", new byte[0], ZooDefs.Ids.OPEN_ACL_UNSAFE,
+                CreateMode.EPHEMERAL);
+        assertNotNull("Ephemeral node has not been created", zk.exists("/srestore", null));
+
+        zk.disconnect();
+        zk.close();
+
+        zk = createClient();
+        assertNotNull("Ephemeral node should be present when session is restored", zk.exists("/srestore", null));
+    }
+
+    /**
+     * Make sure ephemerals are kept when server restarts.
+     */
+    @Test
+    public void testSessionSurviveServerRestart() throws Exception {
+        zk.create("/sdeath", new byte[0], ZooDefs.Ids.OPEN_ACL_UNSAFE,
+                CreateMode.EPHEMERAL);
+        assertNotNull("Ephemeral node has not been created", zk.exists("/sdeath", null));
+
+        zk.disconnect();
+        stopServer();
+        startServer();
+        zk = createClient();
+
+        assertNotNull("Ephemeral node should be present when server restarted", zk.exists("/sdeath", null));
+    }
+}