You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@zookeeper.apache.org by ma...@apache.org on 2012/09/17 07:03:31 UTC

svn commit: r1386472 - in /zookeeper/branches/branch-3.4: ./ src/java/main/org/apache/zookeeper/server/ src/java/main/org/apache/zookeeper/server/quorum/ src/java/test/org/apache/zookeeper/test/

Author: mahadev
Date: Mon Sep 17 05:03:30 2012
New Revision: 1386472

URL: http://svn.apache.org/viewvc?rev=1386472&view=rev
Log:
ZOOKEEPER-1361. Leader.lead iterates over 'learners' set without proper synchronisation. (Henry Robinson via mahadev)

Modified:
    zookeeper/branches/branch-3.4/CHANGES.txt
    zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NIOServerCnxn.java
    zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NettyServerCnxn.java
    zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/Leader.java
    zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/LeaderBean.java
    zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/QuorumPeer.java
    zookeeper/branches/branch-3.4/src/java/test/org/apache/zookeeper/test/QuorumTest.java

Modified: zookeeper/branches/branch-3.4/CHANGES.txt
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/CHANGES.txt?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/CHANGES.txt (original)
+++ zookeeper/branches/branch-3.4/CHANGES.txt Mon Sep 17 05:03:30 2012
@@ -135,6 +135,9 @@ IMPROVEMENTS:
   ZOOKEEPER-1437. Client uses session before SASL authentication complete.
   (Eugene Koontz via mahadev)
 
+  ZOOKEEPER-1361. Leader.lead iterates over 'learners' set without proper
+  synchronisation. (Henry Robinson via mahadev)
+
 Release 3.4.3 - 2012-02-06
 
 Backward compatible changes:

Modified: zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NIOServerCnxn.java
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NIOServerCnxn.java?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NIOServerCnxn.java (original)
+++ zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NIOServerCnxn.java Mon Sep 17 05:03:30 2012
@@ -771,9 +771,9 @@ public class NIOServerCnxn extends Serve
             if(stats.getServerState().equals("leader")) {
                 Leader leader = ((LeaderZooKeeperServer)zkServer).getLeader();
 
-                print("followers", leader.learners.size());
-                print("synced_followers", leader.forwardingFollowers.size());
-                print("pending_syncs", leader.pendingSyncs.size());
+                print("followers", leader.getLearners().size());
+                print("synced_followers", leader.getForwardingFollowers().size());
+                print("pending_syncs", leader.getNumPendingSyncs());
             }
         }
 

Modified: zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NettyServerCnxn.java
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NettyServerCnxn.java?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NettyServerCnxn.java (original)
+++ zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/NettyServerCnxn.java Mon Sep 17 05:03:30 2012
@@ -582,9 +582,9 @@ public class NettyServerCnxn extends Ser
             if(stats.getServerState().equals("leader")) {
                 Leader leader = ((LeaderZooKeeperServer)zkServer).getLeader();
 
-                print("followers", leader.learners.size());
-                print("synced_followers", leader.forwardingFollowers.size());
-                print("pending_syncs", leader.pendingSyncs.size());
+                print("followers", leader.getLearners().size());
+                print("synced_followers", leader.getForwardingFollowers().size());
+                print("pending_syncs", leader.getNumPendingSyncs());
             }
         }
 

Modified: zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/Leader.java
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/Leader.java?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/Leader.java (original)
+++ zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/Leader.java Mon Sep 17 05:03:30 2012
@@ -32,10 +32,10 @@ import java.util.HashMap;
 import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
+import java.util.concurrent.atomic.AtomicLong;
 import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ConcurrentLinkedQueue;
 import java.util.concurrent.ConcurrentMap;
-import java.util.concurrent.atomic.AtomicLong;
 
 import org.apache.jute.BinaryOutputArchive;
 import org.apache.zookeeper.server.FinalRequestProcessor;
@@ -79,22 +79,66 @@ public class Leader {
     LearnerCnxAcceptor cnxAcceptor;
     
     // list of all the followers
-    public final HashSet<LearnerHandler> learners =
+    private final HashSet<LearnerHandler> learners =
         new HashSet<LearnerHandler>();
 
-    // list of followers that are ready to follow (i.e synced with the leader)    
-    public final HashSet<LearnerHandler> forwardingFollowers =
+    /**
+     * Returns a copy of the current learner snapshot
+     */
+    public List<LearnerHandler> getLearners() {
+        synchronized (learners) {
+            return new ArrayList<LearnerHandler>(learners);
+        }
+    }
+
+    // list of followers that are ready to follow (i.e synced with the leader)
+    private final HashSet<LearnerHandler> forwardingFollowers =
         new HashSet<LearnerHandler>();
     
-    protected final HashSet<LearnerHandler> observingLearners =
+    /**
+     * Returns a copy of the current forwarding follower snapshot
+     */
+    public List<LearnerHandler> getForwardingFollowers() {
+        synchronized (forwardingFollowers) {
+            return new ArrayList<LearnerHandler>(forwardingFollowers);
+        }
+    }
+
+    private void addForwardingFollower(LearnerHandler lh) {
+        synchronized (forwardingFollowers) {
+            forwardingFollowers.add(lh);
+        }
+    }
+
+    private final HashSet<LearnerHandler> observingLearners =
         new HashSet<LearnerHandler>();
         
-    //Pending sync requests
-    public final HashMap<Long,List<LearnerSyncRequest>> pendingSyncs =
+    /**
+     * Returns a copy of the current observer snapshot
+     */
+    public List<LearnerHandler> getObservingLearners() {
+        synchronized (observingLearners) {
+            return new ArrayList<LearnerHandler>(observingLearners);
+        }
+    }
+
+    private void addObserverLearnerHandler(LearnerHandler lh) {
+        synchronized (observingLearners) {
+            observingLearners.add(lh);
+        }
+    }
+
+    // Pending sync requests. Must access under 'this' lock.
+    private final HashMap<Long,List<LearnerSyncRequest>> pendingSyncs =
         new HashMap<Long,List<LearnerSyncRequest>>();
     
+    synchronized public int getNumPendingSyncs() {
+        return pendingSyncs.size();
+    }
+
     //Follower counter
     final AtomicLong followerCounter = new AtomicLong(-1);
+
     /**
      * Adds peer to the leader.
      * 
@@ -119,6 +163,9 @@ public class Leader {
         synchronized (learners) {
             learners.remove(peer);
         }
+        synchronized (observingLearners) {
+            observingLearners.remove(peer);
+        }
     }
 
     boolean isLearnerSynced(LearnerHandler peer){
@@ -350,9 +397,11 @@ public class Leader {
                     
                     shutdown("Waiting for a quorum of followers, only synced with: " + ackToString);
                     HashSet<Long> followerSet = new HashSet<Long>();
-                    for(LearnerHandler f : learners)
+
+                    for(LearnerHandler f : getLearners()) {
                         followerSet.add(f.getSid());
-                    
+                    }
+
                     if (self.getQuorumVerifier().containsQuorum(followerSet)) {
                     //if (followers.size() >= self.quorumPeers.size() / 2) {
                         LOG.warn("Enough followers present. "+
@@ -405,16 +454,16 @@ public class Leader {
 
                 // lock on the followers when we use it.
                 syncedSet.add(self.getId());
-                synchronized (learners) {
-                    for (LearnerHandler f : learners) {
-                        // Synced set is used to check we have a supporting quorum, so only
-                        // PARTICIPANT, not OBSERVER, learners should be used
-                        if (f.synced() && f.getLearnerType() == LearnerType.PARTICIPANT) {
-                            syncedSet.add(f.getSid());
-                        }
-                        f.ping();
+
+                for (LearnerHandler f : getLearners()) {
+                    // Synced set is used to check we have a supporting quorum, so only
+                    // PARTICIPANT, not OBSERVER, learners should be used
+                    if (f.synced() && f.getLearnerType() == LearnerType.PARTICIPANT) {
+                        syncedSet.add(f.getSid());
                     }
+                    f.ping();
                 }
+
               if (!tickSkip && !self.getQuorumVerifier().containsQuorum(syncedSet)) {
                 //if (!tickSkip && syncedCount < self.quorumPeers.size() / 2) {
                     // Lost quorum, shutdown
@@ -626,10 +675,8 @@ public class Leader {
      * send a packet to all observers     
      */
     void sendObserverPacket(QuorumPacket qp) {        
-        synchronized(observingLearners) {
-            for (LearnerHandler f : observingLearners) {                
-                f.queuePacket(qp);
-            }
+        for (LearnerHandler f : getObservingLearners()) {
+            f.queuePacket(qp);
         }
     }
 
@@ -789,13 +836,9 @@ public class Leader {
             }
         }
         if (handler.getLearnerType() == LearnerType.PARTICIPANT) {
-            synchronized (forwardingFollowers) {
-                forwardingFollowers.add(handler);
-            }
+            addForwardingFollower(handler);
         } else {
-            synchronized (observingLearners) {
-                observingLearners.add(handler);
-            }
+            addObserverLearnerHandler(handler);
         }
                 
         return lastProposed;

Modified: zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/LeaderBean.java
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/LeaderBean.java?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/LeaderBean.java (original)
+++ zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/LeaderBean.java Mon Sep 17 05:03:30 2012
@@ -44,7 +44,7 @@ public class LeaderBean extends ZooKeepe
     
     public String followerInfo() {
         StringBuilder sb = new StringBuilder();
-        for (LearnerHandler handler : leader.learners) {
+        for (LearnerHandler handler : leader.getLearners()) {
             sb.append(handler.toString()).append("\n");
         }
         return sb.toString();

Modified: zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/QuorumPeer.java
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/QuorumPeer.java?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/QuorumPeer.java (original)
+++ zookeeper/branches/branch-3.4/src/java/main/org/apache/zookeeper/server/quorum/QuorumPeer.java Mon Sep 17 05:03:30 2012
@@ -855,12 +855,8 @@ public class QuorumPeer extends Thread i
         List<String> l = new ArrayList<String>();
         synchronized (this) {
             if (leader != null) {
-                synchronized (leader.learners) {
-                    for (LearnerHandler fh :
-                        leader.learners)
-                    {
-                        if (fh.getSocket() == null)
-                            continue;
+                for (LearnerHandler fh : leader.getLearners()) {
+                    if (fh.getSocket() != null) {
                         String s = fh.getSocket().getRemoteSocketAddress().toString();
                         if (leader.isLearnerSynced(fh))
                             s += "*";

Modified: zookeeper/branches/branch-3.4/src/java/test/org/apache/zookeeper/test/QuorumTest.java
URL: http://svn.apache.org/viewvc/zookeeper/branches/branch-3.4/src/java/test/org/apache/zookeeper/test/QuorumTest.java?rev=1386472&r1=1386471&r2=1386472&view=diff
==============================================================================
--- zookeeper/branches/branch-3.4/src/java/test/org/apache/zookeeper/test/QuorumTest.java (original)
+++ zookeeper/branches/branch-3.4/src/java/test/org/apache/zookeeper/test/QuorumTest.java Mon Sep 17 05:03:30 2012
@@ -151,8 +151,7 @@ public class QuorumTest extends ZKTestCa
                 }
             }, null);
         }
-        ArrayList<LearnerHandler> fhs = new ArrayList<LearnerHandler>(leader.forwardingFollowers);
-        for(LearnerHandler f: fhs) {
+        for(LearnerHandler f : leader.getForwardingFollowers()) {
             f.getSocket().shutdownInput();
         }
         for(int i = 0; i < 5000; i++) {