You are viewing a plain text version of this content. The canonical link for it is here.
Posted to solr-commits@lucene.apache.org by ma...@apache.org on 2010/02/17 22:13:54 UTC

svn commit: r911177 - in /lucene/solr/branches/cloud/src: common/org/apache/solr/common/cloud/ java/org/apache/solr/cloud/ java/org/apache/solr/core/ solrj/org/apache/solr/client/solrj/impl/

Author: markrmiller
Date: Wed Feb 17 21:13:54 2010
New Revision: 911177

URL: http://svn.apache.org/viewvc?rev=911177&view=rev
Log:
refactor, remove nocommit, allow solrj client to add collection watches as well

Modified:
    lucene/solr/branches/cloud/src/common/org/apache/solr/common/cloud/ZkStateReader.java
    lucene/solr/branches/cloud/src/java/org/apache/solr/cloud/ZkController.java
    lucene/solr/branches/cloud/src/java/org/apache/solr/core/CoreContainer.java
    lucene/solr/branches/cloud/src/solrj/org/apache/solr/client/solrj/impl/CloudSolrServer.java

Modified: lucene/solr/branches/cloud/src/common/org/apache/solr/common/cloud/ZkStateReader.java
URL: http://svn.apache.org/viewvc/lucene/solr/branches/cloud/src/common/org/apache/solr/common/cloud/ZkStateReader.java?rev=911177&r1=911176&r2=911177&view=diff
==============================================================================
--- lucene/solr/branches/cloud/src/common/org/apache/solr/common/cloud/ZkStateReader.java (original)
+++ lucene/solr/branches/cloud/src/common/org/apache/solr/common/cloud/ZkStateReader.java Wed Feb 17 21:13:54 2010
@@ -32,6 +32,7 @@
 import org.apache.zookeeper.KeeperException;
 import org.apache.zookeeper.WatchedEvent;
 import org.apache.zookeeper.Watcher;
+import org.apache.zookeeper.Watcher.Event.EventType;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -159,7 +160,7 @@
 
   }
   
-  public void addShardZkNodeWatches() throws KeeperException, InterruptedException {
+  public void makeShardZkNodeWatches() throws KeeperException, InterruptedException {
     CloudState cloudState = getCloudState();
     Set<String> knownCollections = cloudState.getCollections();
     
@@ -172,7 +173,7 @@
           public void process(WatchedEvent event) {
             log.info("Detected changed ShardId in collection:" + collection);
             try {
-              addShardsWatches(collection);
+              makeShardsWatches(collection);
               updateCloudState(false);
             } catch (KeeperException e) {
               log.error("", e);
@@ -216,7 +217,7 @@
     }
   }
   
-  public void addShardsWatches(final String collection) throws KeeperException,
+  public void makeShardsWatches(final String collection) throws KeeperException,
       InterruptedException {
     if (zkClient.exists(COLLECTIONS_ZKNODE + "/" + collection + SHARDS_ZKNODE)) {
       List<String> shardIds = zkClient.getChildren(COLLECTIONS_ZKNODE + "/"
@@ -260,10 +261,14 @@
     }
   }
   
-  public void addShardsWatches() throws KeeperException, InterruptedException {
+  /**
+   * @throws KeeperException
+   * @throws InterruptedException
+   */
+  public void makeShardsWatches() throws KeeperException, InterruptedException {
     List<String> collections = zkClient.getChildren(COLLECTIONS_ZKNODE, null);
     for (final String collection : collections) {
-      addShardsWatches(collection);
+      makeShardsWatches(collection);
     }
   }
   
@@ -291,4 +296,68 @@
       }
     }
   }
+
+  public void makeCollectionsNodeWatches() throws KeeperException, InterruptedException {
+    log.info("Start watching collections zk node for changes");
+    zkClient.getChildren(ZkStateReader.COLLECTIONS_ZKNODE, new Watcher(){
+
+      public void process(WatchedEvent event) {
+          try {
+            log.info("Detected a new or removed collection");
+            synchronized (getUpdateLock()) {
+              makeShardZkNodeWatches();
+              updateCloudState(false);
+            }
+            // re-watch
+            zkClient.getChildren(event.getPath(), this);
+          } catch (KeeperException e) {
+            log.error("", e);
+            throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
+                "", e);
+          } catch (InterruptedException e) {
+            // Restore the interrupted status
+            Thread.currentThread().interrupt();
+            log.error("", e);
+            throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
+                "", e);
+          } catch (IOException e) {
+            log.error("", e);
+            throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
+                "", e);
+          }
+
+      }});
+    
+    zkClient.exists(ZkStateReader.COLLECTIONS_ZKNODE, new Watcher(){
+
+      public void process(WatchedEvent event) {
+        if(event.getType() !=  EventType.NodeDataChanged) {
+          return;
+        }
+        log.info("Notified of CloudState change");
+        try {
+          synchronized (getUpdateLock()) {
+            makeShardZkNodeWatches();
+            updateCloudState(false);
+          }
+          zkClient.exists(ZkStateReader.COLLECTIONS_ZKNODE, this);
+        } catch (KeeperException e) {
+          log.error("", e);
+          throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
+              "", e);
+        } catch (InterruptedException e) {
+          // Restore the interrupted status
+          Thread.currentThread().interrupt();
+          log.error("", e);
+          throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
+              "", e);
+        } catch (IOException e) {
+          log.error("", e);
+          throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
+              "", e);
+        }
+        
+      }});
+    
+  }
 }

Modified: lucene/solr/branches/cloud/src/java/org/apache/solr/cloud/ZkController.java
URL: http://svn.apache.org/viewvc/lucene/solr/branches/cloud/src/java/org/apache/solr/cloud/ZkController.java?rev=911177&r1=911176&r2=911177&view=diff
==============================================================================
--- lucene/solr/branches/cloud/src/java/org/apache/solr/cloud/ZkController.java (original)
+++ lucene/solr/branches/cloud/src/java/org/apache/solr/cloud/ZkController.java Wed Feb 17 21:13:54 2010
@@ -286,6 +286,7 @@
       
       createEphemeralLiveNode();
       setUpCollectionsNode();
+      zkStateReader.makeCollectionsNodeWatches();
       
     } catch (IOException e) {
       log.error("", e);
@@ -548,66 +549,6 @@
           "", e);
     }
     
-    log.info("Start watching collections zk node for changes");
-    zkClient.getChildren(ZkStateReader.COLLECTIONS_ZKNODE, new Watcher(){
-
-      public void process(WatchedEvent event) {
-          try {
-            log.info("Detected a new or removed collection");
-            synchronized (zkStateReader.getUpdateLock()) {
-              zkStateReader.addShardZkNodeWatches();
-              zkStateReader.updateCloudState(false);
-            }
-            // re-watch
-            zkClient.getChildren(event.getPath(), this);
-          } catch (KeeperException e) {
-            log.error("", e);
-            throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
-                "", e);
-          } catch (InterruptedException e) {
-            // Restore the interrupted status
-            Thread.currentThread().interrupt();
-            log.error("", e);
-            throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
-                "", e);
-          } catch (IOException e) {
-            log.error("", e);
-            throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
-                "", e);
-          }
-
-      }});
-    
-    zkClient.exists(ZkStateReader.COLLECTIONS_ZKNODE, new Watcher(){
-
-      public void process(WatchedEvent event) {
-        if(event.getType() !=  EventType.NodeDataChanged) {
-          return;
-        }
-        log.info("Notified of CloudState change");
-        try {
-          synchronized (zkStateReader.getUpdateLock()) {
-            zkStateReader.addShardZkNodeWatches();
-            zkStateReader.updateCloudState(false);
-          }
-          zkClient.exists(ZkStateReader.COLLECTIONS_ZKNODE, this);
-        } catch (KeeperException e) {
-          log.error("", e);
-          throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
-              "", e);
-        } catch (InterruptedException e) {
-          // Restore the interrupted status
-          Thread.currentThread().interrupt();
-          log.error("", e);
-          throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
-              "", e);
-        } catch (IOException e) {
-          log.error("", e);
-          throw new ZooKeeperException(SolrException.ErrorCode.SERVER_ERROR,
-              "", e);
-        }
-        
-      }});
   }
 
   public void createCollectionZkNode(CloudDescriptor cd) throws KeeperException, InterruptedException, IOException {

Modified: lucene/solr/branches/cloud/src/java/org/apache/solr/core/CoreContainer.java
URL: http://svn.apache.org/viewvc/lucene/solr/branches/cloud/src/java/org/apache/solr/core/CoreContainer.java?rev=911177&r1=911176&r2=911177&view=diff
==============================================================================
--- lucene/solr/branches/cloud/src/java/org/apache/solr/core/CoreContainer.java (original)
+++ lucene/solr/branches/cloud/src/java/org/apache/solr/core/CoreContainer.java Wed Feb 17 21:13:54 2010
@@ -431,7 +431,7 @@
     if(zkController != null) {
       try {
         synchronized (zkController.getZkStateReader().getUpdateLock()) {
-          zkController.getZkStateReader().addShardZkNodeWatches();
+          zkController.getZkStateReader().makeShardZkNodeWatches();
           zkController.getZkStateReader().updateCloudState(true);
         }
       } catch (InterruptedException e) {

Modified: lucene/solr/branches/cloud/src/solrj/org/apache/solr/client/solrj/impl/CloudSolrServer.java
URL: http://svn.apache.org/viewvc/lucene/solr/branches/cloud/src/solrj/org/apache/solr/client/solrj/impl/CloudSolrServer.java?rev=911177&r1=911176&r2=911177&view=diff
==============================================================================
--- lucene/solr/branches/cloud/src/solrj/org/apache/solr/client/solrj/impl/CloudSolrServer.java (original)
+++ lucene/solr/branches/cloud/src/solrj/org/apache/solr/client/solrj/impl/CloudSolrServer.java Wed Feb 17 21:13:54 2010
@@ -77,8 +77,8 @@
       if (zkStateReader != null) return;
       try {
         ZkStateReader zk = new ZkStateReader(zkHost, zkConnectTimeout, zkClientTimeout);
-        // nocommit : deal with other watches
-        zk.addShardZkNodeWatches();
+        zk.makeCollectionsNodeWatches();
+        zk.makeShardZkNodeWatches();
         zk.updateCloudState(true);
         zkStateReader = zk;
       } catch (InterruptedException e) {