You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@lucene.apache.org by ma...@apache.org on 2010/10/13 19:01:33 UTC

svn commit: r1022188 [3/4] - in /lucene/dev/trunk/solr: ./ lib/ src/common/org/apache/solr/common/cloud/ src/common/org/apache/solr/common/params/ src/java/org/apache/solr/cloud/ src/java/org/apache/solr/core/ src/java/org/apache/solr/handler/admin/ sr...

Modified: lucene/dev/trunk/solr/src/solrj/org/apache/solr/client/solrj/impl/LBHttpSolrServer.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/solrj/org/apache/solr/client/solrj/impl/LBHttpSolrServer.java?rev=1022188&r1=1022187&r2=1022188&view=diff
==============================================================================
--- lucene/dev/trunk/solr/src/solrj/org/apache/solr/client/solrj/impl/LBHttpSolrServer.java (original)
+++ lucene/dev/trunk/solr/src/solrj/org/apache/solr/client/solrj/impl/LBHttpSolrServer.java Wed Oct 13 17:01:13 2010
@@ -18,6 +18,9 @@ package org.apache.solr.client.solrj.imp
 
 import org.apache.commons.httpclient.HttpClient;
 import org.apache.commons.httpclient.MultiThreadedHttpConnectionManager;
+import org.apache.commons.httpclient.DefaultMethodRetryHandler;
+import org.apache.commons.httpclient.DefaultHttpMethodRetryHandler;
+import org.apache.commons.httpclient.params.HttpMethodParams;
 import org.apache.solr.client.solrj.*;
 import org.apache.solr.client.solrj.response.QueryResponse;
 import org.apache.solr.common.util.NamedList;
@@ -27,26 +30,24 @@ import java.io.IOException;
 import java.lang.ref.WeakReference;
 import java.net.MalformedURLException;
 import java.net.URL;
-import java.util.concurrent.CopyOnWriteArrayList;
-import java.util.concurrent.Executors;
-import java.util.concurrent.ScheduledExecutorService;
-import java.util.concurrent.TimeUnit;
+import java.util.concurrent.*;
 import java.util.concurrent.atomic.AtomicInteger;
 import java.util.concurrent.locks.ReentrantLock;
+import java.util.*;
 
 /**
- * LBHttpSolrServer or "LoadBalanced HttpSolrServer" is just a wrapper to CommonsHttpSolrServer. This is useful when you
+ * LBHttpSolrServer or "LoadBalanced HttpSolrServer" is a load balancing wrapper to CommonsHttpSolrServer. This is useful when you
  * have multiple SolrServers and the requests need to be Load Balanced among them. This should <b>NOT</b> be used for
  * indexing. Also see the <a href="http://wiki.apache.org/solr/LBHttpSolrServer">wiki</a> page.
  * <p/>
  * It offers automatic failover when a server goes down and it detects when the server comes back up.
  * <p/>
- * Load balancing is done using a simple roundrobin on the list of servers.
+ * Load balancing is done using a simple round-robin on the list of servers.
  * <p/>
  * If a request to a server fails by an IOException due to a connection timeout or read timeout then the host is taken
  * off the list of live servers and moved to a 'dead server list' and the request is resent to the next live server.
  * This process is continued till it tries all the live servers. If atleast one server is alive, the request succeeds,
- * andif not it fails.
+ * and if not it fails.
  * <blockquote><pre>
  * SolrServer lbHttpSolrServer = new LBHttpSolrServer("http://host1:8080/solr/","http://host2:8080/solr","http://host2:8080/solr");
  * //or if you wish to pass the HttpClient do as follows
@@ -57,22 +58,33 @@ import java.util.concurrent.locks.Reentr
  * This interval can be set using {@link #setAliveCheckInterval} , the default is set to one minute.
  * <p/>
  * <b>When to use this?</b><br/> This can be used as a software load balancer when you do not wish to setup an external
- * load balancer. The code is relatively new and the API is currently experimental. Alternatives to this code are to use
+ * load balancer. Alternatives to this code are to use
  * a dedicated hardware load balancer or using Apache httpd with mod_proxy_balancer as a load balancer. See <a
  * href="http://en.wikipedia.org/wiki/Load_balancing_(computing)">Load balancing on Wikipedia</a>
  *
  * @since solr 1.4
  */
 public class LBHttpSolrServer extends SolrServer {
-  private final CopyOnWriteArrayList<ServerWrapper> aliveServers = new CopyOnWriteArrayList<ServerWrapper>();
-  private final CopyOnWriteArrayList<ServerWrapper> zombieServers = new CopyOnWriteArrayList<ServerWrapper>();
+
+
+  // keys to the maps are currently of the form "http://localhost:8983/solr"
+  // which should be equivalent to CommonsHttpSolrServer.getBaseURL()
+  private final Map<String, ServerWrapper> aliveServers = new LinkedHashMap<String, ServerWrapper>();
+  // access to aliveServers should be synchronized on itself
+  
+  private final Map<String, ServerWrapper> zombieServers = new ConcurrentHashMap<String, ServerWrapper>();
+
+  // changes to aliveServers are reflected in this array, no need to synchronize
+  private volatile ServerWrapper[] aliveServerList = new ServerWrapper[0];
+
+
   private ScheduledExecutorService aliveCheckExecutor;
 
   private HttpClient httpClient;
   private final AtomicInteger counter = new AtomicInteger(-1);
 
-  private ReentrantLock checkLock = new ReentrantLock();
   private static final SolrQuery solrQuery = new SolrQuery("*:*");
+  private static final BinaryResponseParser binaryParser = new BinaryResponseParser();
 
   static {
     solrQuery.setRows(0);
@@ -81,8 +93,13 @@ public class LBHttpSolrServer extends So
   private static class ServerWrapper {
     final CommonsHttpSolrServer solrServer;
 
-    // Used only by the thread in aliveCheckExecutor
-    long lastUsed, lastChecked;
+    long lastUsed;     // last time used for a real request
+    long lastChecked;  // last time checked for liveness
+
+    // "standard" servers are used by default.  They normally live in the alive list
+    // and move to the zombie list when unavailable.  When they become available again,
+    // they move back to the alive list.
+    boolean standard = true;
 
     int failedPings = 0;
 
@@ -93,35 +110,225 @@ public class LBHttpSolrServer extends So
     public String toString() {
       return solrServer.getBaseURL();
     }
+
+    public String getKey() {
+      return solrServer.getBaseURL();
+    }
+
+    @Override
+    public int hashCode() {
+      return this.getKey().hashCode();
+    }
+
+    @Override
+    public boolean equals(Object obj) {
+      if (this == obj) return true;
+      if (!(obj instanceof ServerWrapper)) return false;
+      return this.getKey().equals(((ServerWrapper)obj).getKey());
+    }
+  }
+
+  public static class Req {
+    protected SolrRequest request;
+    protected List<String> servers;
+    protected int numDeadServersToTry;
+
+    public Req(SolrRequest request, List<String> servers) {
+      this.request = request;
+      this.servers = servers;
+      this.numDeadServersToTry = servers.size();
+    }
+
+    public SolrRequest getRequest() {
+      return request;
+    }
+    public List<String> getServers() {
+      return servers;
+    }
+
+    /** @return the number of dead servers to try if there are no live servers left */
+    public int getNumDeadServersToTry() {
+      return numDeadServersToTry;
+    }
+
+    /** @return The number of dead servers to try if there are no live servers left.
+     * Defaults to the number of servers in this request. */
+    public void setNumDeadServersToTry(int numDeadServersToTry) {
+      this.numDeadServersToTry = numDeadServersToTry;
+    }
+  }
+
+  public static class Rsp {
+    protected String server;
+    protected NamedList<Object> rsp;
+
+    /** The response from the server */
+    public NamedList<Object> getResponse() {
+      return rsp;
+    }
+
+    /** The server that returned the response */
+    public String getServer() {
+      return server;
+    }
   }
 
   public LBHttpSolrServer(String... solrServerUrls) throws MalformedURLException {
     this(new HttpClient(new MultiThreadedHttpConnectionManager()), solrServerUrls);
+
+    DefaultHttpMethodRetryHandler retryhandler = new DefaultHttpMethodRetryHandler(0, false);
+    httpClient.getParams().setParameter(HttpMethodParams.RETRY_HANDLER, retryhandler);
   }
 
+  /** The provided httpClient should use a multi-threaded connection manager */ 
   public LBHttpSolrServer(HttpClient httpClient, String... solrServerUrl)
           throws MalformedURLException {
     this(httpClient, new BinaryResponseParser(), solrServerUrl);
   }
 
+  /** The provided httpClient should use a multi-threaded connection manager */  
   public LBHttpSolrServer(HttpClient httpClient, ResponseParser parser, String... solrServerUrl)
           throws MalformedURLException {
     this.httpClient = httpClient;
     for (String s : solrServerUrl) {
-      aliveServers.add(new ServerWrapper(new CommonsHttpSolrServer(s, httpClient, parser)));
+      ServerWrapper wrapper = new ServerWrapper(makeServer(s));
+      aliveServers.put(wrapper.getKey(), wrapper);
     }
+    updateAliveList();
   }
 
-  public void addSolrServer(String server) throws MalformedURLException {
-    CommonsHttpSolrServer solrServer = new CommonsHttpSolrServer(server, httpClient);
-    checkLock.lock();
-    try {
-      aliveServers.add(new ServerWrapper(solrServer));
-    } finally {
-      checkLock.unlock();
+  public static String normalize(String server) {
+    if (server.endsWith("/"))
+      server = server.substring(0, server.length() - 1);
+    return server;
+  }
+
+  protected CommonsHttpSolrServer makeServer(String server) throws MalformedURLException {
+    return new CommonsHttpSolrServer(server, httpClient, binaryParser);
+  }
+
+
+
+  /**
+   * Tries to query a live server from the list provided in Req. Servers in the dead pool are skipped.
+   * If a request fails due to an IOException, the server is moved to the dead pool for a certain period of
+   * time, or until a test request on that server succeeds.
+   *
+   * Servers are queried in the exact order given (except servers currently in the dead pool are skipped).
+   * If no live servers from the provided list remain to be tried, a number of previously skipped dead servers will be tried.
+   * Req.getNumDeadServersToTry() controls how many dead servers will be tried.
+   *
+   * If no live servers are found a SolrServerException is thrown.
+   *
+   * @param req contains both the request as well as the list of servers to query
+   *
+   * @return the result of the request
+   *
+   * @throws SolrServerException
+   * @throws IOException
+   */
+  public Rsp request(Req req) throws SolrServerException, IOException {
+    Rsp rsp = new Rsp();
+    Exception ex = null;
+
+    List<ServerWrapper> skipped = new ArrayList<ServerWrapper>(req.getNumDeadServersToTry());
+
+    for (String serverStr : req.getServers()) {
+      serverStr = normalize(serverStr);
+      // if the server is currently a zombie, just skip to the next one
+      ServerWrapper wrapper = zombieServers.get(serverStr);
+      if (wrapper != null) {
+        // System.out.println("ZOMBIE SERVER QUERIED: " + serverStr);
+        if (skipped.size() < req.getNumDeadServersToTry())
+          skipped.add(wrapper);
+        continue;
+      }
+      rsp.server = serverStr;
+      CommonsHttpSolrServer server = makeServer(serverStr);
+
+      try {
+        rsp.rsp = server.request(req.getRequest());
+        return rsp; // SUCCESS
+      } catch (SolrException e) {
+        // Server is alive but the request was malformed or invalid
+        throw e;
+      } catch (SolrServerException e) {
+        if (e.getRootCause() instanceof IOException) {
+          ex = e;
+          wrapper = new ServerWrapper(server);
+          wrapper.lastUsed = System.currentTimeMillis();
+          wrapper.standard = false;
+          zombieServers.put(wrapper.getKey(), wrapper);
+          startAliveCheckExecutor();
+        } else {
+          throw e;
+        }
+      } catch (Exception e) {
+        throw new SolrServerException(e);
+      }
+    }
+
+    // try the servers we previously skipped
+    for (ServerWrapper wrapper : skipped) {
+      try {
+        rsp.rsp = wrapper.solrServer.request(req.getRequest());
+        zombieServers.remove(wrapper.getKey());
+        return rsp; // SUCCESS
+      } catch (SolrException e) {
+        // Server is alive but the request was malformed or invalid
+        zombieServers.remove(wrapper.getKey());
+        throw e;
+      } catch (SolrServerException e) {
+        if (e.getRootCause() instanceof IOException) {
+          ex = e;
+          // already a zombie, no need to re-add
+        } else {
+          throw e;
+        }
+      } catch (Exception e) {
+        throw new SolrServerException(e);
+      }
+    }
+
+
+    if (ex == null) {
+      throw new SolrServerException("No live SolrServers available to handle this request");
+    } else {
+      throw new SolrServerException("No live SolrServers available to handle this request", ex);
+    }
+
+  }  
+
+
+
+  private void updateAliveList() {
+    synchronized (aliveServers) {
+      aliveServerList = aliveServers.values().toArray(new ServerWrapper[aliveServers.size()]);
+    }
+  }
+
+  private ServerWrapper removeFromAlive(String key) {
+    synchronized (aliveServers) {
+      ServerWrapper wrapper = aliveServers.remove(key);
+      if (wrapper != null)
+        updateAliveList();
+      return wrapper;
+    }
+  }
+
+  private void addToAlive(ServerWrapper wrapper) {
+    synchronized (aliveServers) {
+      ServerWrapper prev = aliveServers.put(wrapper.getKey(), wrapper);
+      // TODO: warn if there was a previous entry?
+      updateAliveList();
     }
   }
 
+  public void addSolrServer(String server) throws MalformedURLException {
+    CommonsHttpSolrServer solrServer = makeServer(server);
+    addToAlive(new ServerWrapper(solrServer));
+  }
+
   public String removeSolrServer(String server) {
     try {
       server = new URL(server).toExternalForm();
@@ -131,25 +338,11 @@ public class LBHttpSolrServer extends So
     if (server.endsWith("/")) {
       server = server.substring(0, server.length() - 1);
     }
-    this.checkLock.lock();
-    try {
-      for (ServerWrapper serverWrapper : aliveServers) {
-        if (serverWrapper.solrServer.getBaseURL().equals(server)) {
-          aliveServers.remove(serverWrapper);
-          return serverWrapper.solrServer.getBaseURL();
-        }
-      }
-      if (zombieServers.isEmpty()) return null;
 
-      for (ServerWrapper serverWrapper : zombieServers) {
-        if (serverWrapper.solrServer.getBaseURL().equals(server)) {
-          zombieServers.remove(serverWrapper);
-          return serverWrapper.solrServer.getBaseURL();
-        }
-      }
-    } finally {
-      checkLock.unlock();
-    }
+    // there is a small race condition here - if the server is in the process of being moved between
+    // lists, we could fail to remove it.
+    removeFromAlive(server);
+    zombieServers.remove(server);
     return null;
   }
 
@@ -173,9 +366,10 @@ public class LBHttpSolrServer extends So
   }
 
   /**
-   * Tries to query a live server. If no live servers are found it throws a SolrServerException. If the request failed
-   * due to IOException then the live server is moved to dead pool and the request is retried on another live server if
-   * available. If all live servers are exhausted then a SolrServerException is thrown.
+   * Tries to query a live server. A SolrServerException is thrown if all servers are dead.
+   * If the request failed due to IOException then the live server is moved to dead pool and the request is
+   * retried on another live server.  After live servers are exhausted, any servers previously marked as dead
+   * will be tried before failing the request.
    *
    * @param request the SolrRequest.
    *
@@ -186,41 +380,69 @@ public class LBHttpSolrServer extends So
    */
   public NamedList<Object> request(final SolrRequest request)
           throws SolrServerException, IOException {
-    int count = counter.incrementAndGet();
-    int attempts = 0;
-    Exception ex;
-    int startSize = aliveServers.size();
-    while (true) {
-      int size = aliveServers.size();
-      if (size < 1) throw new SolrServerException("No live SolrServers available to handle this request");
-      ServerWrapper solrServer;
+    Exception ex = null;
+    ServerWrapper[] serverList = aliveServerList;
+    
+    int maxTries = serverList.length;
+    Map<String,ServerWrapper> justFailed = null;
+
+    for (int attempts=0; attempts<maxTries; attempts++) {
+      int count = counter.incrementAndGet();      
+      ServerWrapper wrapper = serverList[count % serverList.length];
+      wrapper.lastUsed = System.currentTimeMillis();
+
       try {
-        solrServer = aliveServers.get(count % size);
-      } catch (IndexOutOfBoundsException e) {
-        //this list changes dynamically. so it is expected to get IndexOutOfBoundsException
-        continue;
+        return wrapper.solrServer.request(request);
+      } catch (SolrException e) {
+        // Server is alive but the request was malformed or invalid
+        throw e;
+      } catch (SolrServerException e) {
+        if (e.getRootCause() instanceof IOException) {
+          ex = e;
+          moveAliveToDead(wrapper);
+          if (justFailed == null) justFailed = new HashMap<String,ServerWrapper>();
+          justFailed.put(wrapper.getKey(), wrapper);
+        } else {
+          throw e;
+        }
+      } catch (Exception e) {
+        throw new SolrServerException(e);
       }
+    }
+
+
+    // try other standard servers that we didn't try just now
+    for (ServerWrapper wrapper : zombieServers.values()) {
+      if (wrapper.standard==false || justFailed!=null && justFailed.containsKey(wrapper.getKey())) continue;
       try {
-        return solrServer.solrServer.request(request);
+        NamedList<Object> rsp = wrapper.solrServer.request(request);
+        // remove from zombie list *before* adding to alive to avoid a race that could lose a server
+        zombieServers.remove(wrapper.getKey());
+        addToAlive(wrapper);
+        return rsp;
       } catch (SolrException e) {
         // Server is alive but the request was malformed or invalid
         throw e;
       } catch (SolrServerException e) {
         if (e.getRootCause() instanceof IOException) {
           ex = e;
-          moveAliveToDead(solrServer);
+          // still dead
         } else {
           throw e;
         }
       } catch (Exception e) {
         throw new SolrServerException(e);
       }
-      attempts++;
-      if (attempts >= startSize)
-        throw new SolrServerException("No live SolrServers available to handle this request", ex);
     }
-  }
 
+
+    if (ex == null) {
+      throw new SolrServerException("No live SolrServers available to handle this request");
+    } else {
+      throw new SolrServerException("No live SolrServers available to handle this request", ex);
+    }
+  }
+  
   /**
    * Takes up one dead server and check for aliveness. The check is done in a roundrobin. Each server is checked for
    * aliveness once in 'x' millis where x is decided by the setAliveCheckinterval() or it is defaulted to 1 minute
@@ -229,39 +451,44 @@ public class LBHttpSolrServer extends So
    */
   private void checkAZombieServer(ServerWrapper zombieServer) {
     long currTime = System.currentTimeMillis();
-    checkLock.lock();
     try {
       zombieServer.lastChecked = currTime;
       QueryResponse resp = zombieServer.solrServer.query(solrQuery);
       if (resp.getStatus() == 0) {
-        //server has come back up
-        zombieServer.lastUsed = currTime;
-        zombieServers.remove(zombieServer);
-        aliveServers.add(zombieServer);
-        zombieServer.failedPings = 0;
+        // server has come back up.
+        // make sure to remove from zombies before adding to alive to avoid a race condition
+        // where another thread could mark it down, move it back to zombie, and then we delete
+        // from zombie and lose it forever.
+        ServerWrapper wrapper = zombieServers.remove(zombieServer.getKey());
+        if (wrapper != null) {
+          wrapper.failedPings = 0;
+          if (wrapper.standard) {
+            addToAlive(wrapper);
+          }
+        } else {
+          // something else already moved the server from zombie to alive
+        }
       }
     } catch (Exception e) {
+      //Expected. The server is still down.
       zombieServer.failedPings++;
-      //Expected . The server is still down
-    } finally {
-      checkLock.unlock();
-    }
-  }
 
-  private void moveAliveToDead(ServerWrapper solrServer) {
-    checkLock.lock();
-    try {
-      boolean result = aliveServers.remove(solrServer);
-      if (result) {
-        if (zombieServers.addIfAbsent(solrServer)) {
-          startAliveCheckExecutor();
-        }
+      // If the server doesn't belong in the standard set belonging to this load balancer
+      // then simply drop it after a certain number of failed pings.
+      if (!zombieServer.standard && zombieServer.failedPings >= NONSTANDARD_PING_LIMIT) {
+        zombieServers.remove(zombieServer.getKey());
       }
-    } finally {
-      checkLock.unlock();
     }
   }
 
+  private void moveAliveToDead(ServerWrapper wrapper) {
+    wrapper = removeFromAlive(wrapper.getKey());
+    if (wrapper == null)
+      return;  // another thread already detected the failure and removed it
+    zombieServers.put(wrapper.getKey(), wrapper);
+    startAliveCheckExecutor();
+  }
+
   private int interval = CHECK_INTERVAL;
 
   /**
@@ -279,6 +506,8 @@ public class LBHttpSolrServer extends So
   }
 
   private void startAliveCheckExecutor() {
+    // double-checked locking, but it's OK because we don't *do* anything with aliveCheckExecutor
+    // if it's not null.
     if (aliveCheckExecutor == null) {
       synchronized (this) {
         if (aliveCheckExecutor == null) {
@@ -291,13 +520,13 @@ public class LBHttpSolrServer extends So
     }
   }
 
-  private static Runnable getAliveCheckRunner(final WeakReference<LBHttpSolrServer> lbHttpSolrServer) {
+  private static Runnable getAliveCheckRunner(final WeakReference<LBHttpSolrServer> lbRef) {
     return new Runnable() {
       public void run() {
-        LBHttpSolrServer solrServer = lbHttpSolrServer.get();
-        if (solrServer != null && solrServer.zombieServers != null) {
-          for (ServerWrapper zombieServer : solrServer.zombieServers) {
-            solrServer.checkAZombieServer(zombieServer);
+        LBHttpSolrServer lb = lbRef.get();
+        if (lb != null && lb.zombieServers != null) {
+          for (ServerWrapper zombieServer : lb.zombieServers.values()) {
+            lb.checkAZombieServer(zombieServer);
           }
         }
       }
@@ -317,5 +546,7 @@ public class LBHttpSolrServer extends So
     }
   }
 
+  // defaults
   private static final int CHECK_INTERVAL = 60 * 1000; //1 minute between checks
+  private static final int NONSTANDARD_PING_LIMIT = 5;  // number of times we'll ping dead servers not in the server list
 }

Modified: lucene/dev/trunk/solr/src/test/org/apache/solr/BaseDistributedSearchTestCase.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/BaseDistributedSearchTestCase.java?rev=1022188&r1=1022187&r2=1022188&view=diff
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/BaseDistributedSearchTestCase.java (original)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/BaseDistributedSearchTestCase.java Wed Oct 13 17:01:13 2010
@@ -1,6 +1,20 @@
 package org.apache.solr;
 
+import java.io.File;
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Random;
+import java.util.Set;
+
 import junit.framework.TestCase;
+
 import org.apache.solr.client.solrj.SolrServer;
 import org.apache.solr.client.solrj.SolrServerException;
 import org.apache.solr.client.solrj.embedded.JettySolrRunner;
@@ -17,10 +31,6 @@ import org.junit.Test;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
-import java.io.File;
-import java.io.IOException;
-import java.util.*;
-
 /**
  * Helper base class for distributed search test cases
  *
@@ -44,6 +54,8 @@ public abstract class BaseDistributedSea
   protected List<JettySolrRunner> jettys = new ArrayList<JettySolrRunner>();
   protected String context = "/solr";
   protected String shards;
+  protected String[] shardsArr;
+  protected String[] deadServers = {"does_not_exist_54321.com:33331/solr","localhost:33332/solr"};
   protected File testDir;
   protected SolrServer controlClient;
 
@@ -130,40 +142,79 @@ public abstract class BaseDistributedSea
     super.tearDown();
   }
 
-  private void createServers(int numShards) throws Exception {
-    controlJetty = createJetty(testDir, "control");
+  protected void createServers(int numShards) throws Exception {
+    controlJetty = createJetty(testDir, testDir + "/control/data");
     controlClient = createNewSolrServer(controlJetty.getLocalPort());
 
+    shardsArr = new String[numShards];
     StringBuilder sb = new StringBuilder();
-    for (int i = 1; i <= numShards; i++) {
+    for (int i = 0; i < numShards; i++) {
       if (sb.length() > 0) sb.append(',');
-      JettySolrRunner j = createJetty(testDir, "shard" + i);
+      JettySolrRunner j = createJetty(testDir, testDir + "/shard" + i + "/data");
       jettys.add(j);
       clients.add(createNewSolrServer(j.getLocalPort()));
-      sb.append("localhost:").append(j.getLocalPort()).append(context);
+      String shardStr = "localhost:" + j.getLocalPort() + context;
+      shardsArr[i] = shardStr;
+      sb.append(shardStr);
     }
 
     shards = sb.toString();
   }
 
+
+  protected void setDistributedParams(ModifiableSolrParams params) {
+    params.set("shards", getShardsString());
+  }
+
+  protected String getShardsString() {
+    if (deadServers == null) return shards;
+    
+    StringBuilder sb = new StringBuilder();
+    for (String shard : shardsArr) {
+      if (sb.length() > 0) sb.append(',');
+      int nDeadServers = r.nextInt(deadServers.length+1);
+      if (nDeadServers > 0) {
+        List<String> replicas = new ArrayList<String>(Arrays.asList(deadServers));
+        Collections.shuffle(replicas, r);
+        replicas.add(r.nextInt(nDeadServers+1), shard);
+        for (int i=0; i<nDeadServers+1; i++) {
+          if (i!=0) sb.append('|');
+          sb.append(replicas.get(i));
+        }
+      } else {
+        sb.append(shard);
+      }
+    }
+
+    return sb.toString();
+  }
+
   protected void destroyServers() throws Exception {
     controlJetty.stop();
     for (JettySolrRunner jetty : jettys) jetty.stop();
     clients.clear();
     jettys.clear();
   }
+  
+  public JettySolrRunner createJetty(File baseDir, String dataDir) throws Exception {
+    return createJetty(baseDir, dataDir, null, null);
+  }
 
-  public static JettySolrRunner createJetty(File baseDir, String dataDirName) throws Exception {
-    File subDir = new File(baseDir, dataDirName);
-    subDir.mkdirs();
-    System.setProperty("solr.data.dir", subDir.toString());
-
-    JettySolrRunner jetty = new JettySolrRunner("/solr", 0);
-
+  public JettySolrRunner createJetty(File baseDir, String dataDir, String shardId) throws Exception {
+    return createJetty(baseDir, dataDir, shardId, null);
+  }
+  
+  public JettySolrRunner createJetty(File baseDir, String dataDir, String shardList, String solrConfigOverride) throws Exception {
+    System.setProperty("solr.data.dir", dataDir);
+    JettySolrRunner jetty = new JettySolrRunner("/solr", 0, solrConfigOverride);
+    if(shardList != null) {
+      System.setProperty("shard", shardList);
+    }
     jetty.start();
+    System.clearProperty("shard");
     return jetty;
   }
-
+  
   protected SolrServer createNewSolrServer(int port) {
     try {
       // setup the server...
@@ -230,6 +281,14 @@ public abstract class BaseDistributedSea
     for (SolrServer client : clients) client.commit();
   }
 
+  protected QueryResponse queryServer(ModifiableSolrParams params) throws SolrServerException {
+    // query a random server
+    int which = r.nextInt(clients.size());
+    SolrServer client = clients.get(which);
+    QueryResponse rsp = client.query(params);
+    return rsp;
+  }
+
   protected void query(Object... q) throws Exception {
     final ModifiableSolrParams params = new ModifiableSolrParams();
 
@@ -239,11 +298,9 @@ public abstract class BaseDistributedSea
 
     final QueryResponse controlRsp = controlClient.query(params);
 
-    // query a random server
-    params.set("shards", shards);
-    int which = r.nextInt(clients.size());
-    SolrServer client = clients.get(which);
-    QueryResponse rsp = client.query(params);
+    setDistributedParams(params);
+
+    QueryResponse rsp = queryServer(params);
 
     compareResponses(rsp, controlRsp);
 

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractDistributedZkTestCase.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractDistributedZkTestCase.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractDistributedZkTestCase.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractDistributedZkTestCase.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,92 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.io.File;
+
+import org.apache.solr.BaseDistributedSearchTestCase;
+import org.apache.solr.client.solrj.embedded.JettySolrRunner;
+import org.apache.solr.common.cloud.SolrZkClient;
+import org.apache.solr.core.SolrConfig;
+import org.junit.Before;
+
+public abstract class AbstractDistributedZkTestCase extends BaseDistributedSearchTestCase {
+  private static final boolean DEBUG = false;
+  protected ZkTestServer zkServer;
+
+  @Before
+  @Override
+  public void setUp() throws Exception {
+    super.setUp();
+    log.info("####SETUP_START " + getName());
+    
+    ignoreException("java.nio.channels.ClosedChannelException");
+    
+    String zkDir = testDir.getAbsolutePath() + File.separator
+    + "zookeeper/server1/data";
+    zkServer = new ZkTestServer(zkDir);
+    zkServer.run();
+    
+    System.setProperty("zkHost", zkServer.getZkAddress());
+    
+    AbstractZkTestCase.buildZooKeeper(zkServer.getZkHost(), zkServer.getZkAddress(), "solrconfig.xml", "schema.xml");
+
+    // set some system properties for use by tests
+    System.setProperty("solr.test.sys.prop1", "propone");
+    System.setProperty("solr.test.sys.prop2", "proptwo");
+  }
+  
+  protected void createServers(int numShards) throws Exception {
+    System.setProperty("collection", "control_collection");
+    controlJetty = createJetty(testDir, testDir + "/control/data", "control_shard");
+    System.clearProperty("collection");
+    controlClient = createNewSolrServer(controlJetty.getLocalPort());
+
+    StringBuilder sb = new StringBuilder();
+    for (int i = 1; i <= numShards; i++) {
+      if (sb.length() > 0) sb.append(',');
+      JettySolrRunner j = createJetty(testDir, testDir + "/jetty" + i, "shard" + (i + 2));
+      jettys.add(j);
+      clients.add(createNewSolrServer(j.getLocalPort()));
+      sb.append("localhost:").append(j.getLocalPort()).append(context);
+    }
+
+    shards = sb.toString();
+  }
+
+  @Override
+  public void tearDown() throws Exception {
+    if (DEBUG) {
+      printLayout();
+    }
+    zkServer.shutdown();
+    System.clearProperty("zkHost");
+    System.clearProperty("collection");
+    System.clearProperty("solr.test.sys.prop1");
+    System.clearProperty("solr.test.sys.prop2");
+    super.tearDown();
+    resetExceptionIgnores();
+    SolrConfig.severeErrors.clear();
+  }
+  
+  protected void printLayout() throws Exception {
+    SolrZkClient zkClient = new SolrZkClient(zkServer.getZkHost(), AbstractZkTestCase.TIMEOUT);
+    zkClient.printLayoutToStdOut();
+    zkClient.close();
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractZkTestCase.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractZkTestCase.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractZkTestCase.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/AbstractZkTestCase.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,149 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.io.File;
+
+import org.apache.solr.SolrTestCaseJ4;
+import org.apache.solr.common.cloud.SolrZkClient;
+import org.apache.solr.common.cloud.ZkNodeProps;
+import org.apache.solr.core.CoreContainer;
+import org.apache.solr.core.SolrConfig;
+import org.apache.solr.util.TestHarness;
+import org.apache.zookeeper.CreateMode;
+import org.junit.BeforeClass;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Base test class for ZooKeeper tests.
+ */
+public abstract class AbstractZkTestCase extends SolrTestCaseJ4 {
+
+  static final int TIMEOUT = 10000;
+
+  private static final boolean DEBUG = false;
+
+  protected static Logger log = LoggerFactory
+      .getLogger(AbstractZkTestCase.class);
+
+  protected ZkTestServer zkServer;
+
+  protected String zkDir;
+
+  public AbstractZkTestCase() {
+
+  }
+
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    initCore("solrconfig.xml", "schema12.xml");
+  }
+  
+  @Override
+  public void setUp() throws Exception {
+
+    super.setUp();
+    
+    zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    zkServer = new ZkTestServer(zkDir);
+    zkServer.run();
+    System.setProperty("zkHost", zkServer.getZkAddress());
+    buildZooKeeper(zkServer.getZkHost(), zkServer.getZkAddress(),
+        getSolrConfigFile(), getSchemaFile());
+    
+    log.info("####SETUP_START " + getName());
+
+    dataDir.mkdirs();
+    
+    // set some system properties for use by tests
+    System.setProperty("solr.test.sys.prop1", "propone");
+    System.setProperty("solr.test.sys.prop2", "proptwo");
+    
+    CoreContainer.Initializer init = new CoreContainer.Initializer() {
+      {
+        this.dataDir = AbstractZkTestCase.dataDir.getAbsolutePath();
+      }
+    };
+    
+    h = new TestHarness("", init);
+    lrf = h.getRequestFactory("standard", 0, 20, "version", "2.2");
+    
+    log.info("####SETUP_END " + getName());
+    
+  }
+
+  // static to share with distrib test
+  static void buildZooKeeper(String zkHost, String zkAddress, String config,
+      String schema) throws Exception {
+    SolrZkClient zkClient = new SolrZkClient(zkHost, AbstractZkTestCase.TIMEOUT);
+    zkClient.makePath("/solr");
+    zkClient.close();
+
+    zkClient = new SolrZkClient(zkAddress, AbstractZkTestCase.TIMEOUT);
+
+    ZkNodeProps props = new ZkNodeProps();
+    props.put("configName", "conf1");
+    zkClient.makePath("/collections/collection1", props.store(), CreateMode.PERSISTENT);
+    zkClient.makePath("/collections/collection1/shards", CreateMode.PERSISTENT);
+
+    zkClient.makePath("/collections/control_collection", props.store(), CreateMode.PERSISTENT);
+    zkClient.makePath("/collections/control_collection/shards", CreateMode.PERSISTENT);
+
+    putConfig(zkClient, config);
+    putConfig(zkClient, schema);
+    putConfig(zkClient, "stopwords.txt");
+    putConfig(zkClient, "protwords.txt");
+    putConfig(zkClient, "mapping-ISOLatin1Accent.txt");
+    putConfig(zkClient, "old_synonyms.txt");
+    putConfig(zkClient, "synonyms.txt");
+    
+    zkClient.close();
+  }
+
+  private static void putConfig(SolrZkClient zkConnection, String name)
+      throws Exception {
+    zkConnection.setData("/configs/conf1/" + name, new File("solr"
+        + File.separator + "conf" + File.separator + name));
+  }
+
+  public void tearDown() throws Exception {
+    if (DEBUG) {
+      printLayout(zkServer.getZkHost());
+    }
+    zkServer.shutdown();
+    System.clearProperty("zkHost");
+    System.clearProperty("solr.test.sys.prop1");
+    System.clearProperty("solr.test.sys.prop2");
+    SolrConfig.severeErrors.clear();
+    super.tearDown();
+  }
+
+  private void printLayout(String zkHost) throws Exception {
+    SolrZkClient zkClient = new SolrZkClient(zkHost, AbstractZkTestCase.TIMEOUT);
+    zkClient.printLayoutToStdOut();
+    zkClient.close();
+  }
+
+  static void makeSolrZkNode(String zkHost) throws Exception {
+    SolrZkClient zkClient = new SolrZkClient(zkHost, TIMEOUT);
+    zkClient.makePath("/solr");
+    zkClient.close();
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicDistributedZkTest.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicDistributedZkTest.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicDistributedZkTest.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicDistributedZkTest.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,283 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.net.MalformedURLException;
+
+import org.apache.solr.client.solrj.SolrServerException;
+import org.apache.solr.client.solrj.impl.CloudSolrServer;
+import org.apache.solr.client.solrj.response.QueryResponse;
+import org.apache.solr.common.params.CommonParams;
+import org.apache.solr.common.params.ModifiableSolrParams;
+import org.apache.solr.core.SolrConfig;
+import org.junit.BeforeClass;
+
+/**
+ *
+ */
+public class BasicDistributedZkTest extends AbstractDistributedZkTestCase {
+  
+  private static final String DEFAULT_COLLECTION = "collection1";
+  private static final boolean DEBUG = false;
+  String t1="a_t";
+  String i1="a_si";
+  String nint = "n_i";
+  String tint = "n_ti";
+  String nfloat = "n_f";
+  String tfloat = "n_tf";
+  String ndouble = "n_d";
+  String tdouble = "n_td";
+  String nlong = "n_l";
+  String tlong = "n_tl";
+  String ndate = "n_dt";
+  String tdate = "n_tdt";
+  
+  String oddField="oddField_s";
+  String missingField="ignore_exception__missing_but_valid_field_t";
+  String invalidField="ignore_exception__invalid_field_not_in_schema";
+  
+  public BasicDistributedZkTest() {
+    fixShardCount = true;
+    
+    System.setProperty("CLOUD_UPDATE_DELAY", "0");
+  }
+
+  
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    
+  }
+  
+  @Override
+  protected void setDistributedParams(ModifiableSolrParams params) {
+
+    if (r.nextBoolean()) {
+      // don't set shards, let that be figured out from the cloud state
+      params.set("distrib", "true");
+    } else {
+      // use shard ids rather than physical locations
+      StringBuilder sb = new StringBuilder();
+      for (int i = 0; i < shardCount; i++) {
+        if (i > 0)
+          sb.append(',');
+        sb.append("shard" + (i + 3));
+      }
+      params.set("shards", sb.toString());
+      params.set("distrib", "true");
+    }
+  }
+  
+  @Override
+  public void doTest() throws Exception {
+    del("*:*");
+    indexr(id,1, i1, 100, tlong, 100,t1,"now is the time for all good men"
+            ,"foo_f", 1.414f, "foo_b", "true", "foo_d", 1.414d);
+    indexr(id,2, i1, 50 , tlong, 50,t1,"to come to the aid of their country."
+    );
+    indexr(id,3, i1, 2, tlong, 2,t1,"how now brown cow"
+    );
+    indexr(id,4, i1, -100 ,tlong, 101,t1,"the quick fox jumped over the lazy dog"
+    );
+    indexr(id,5, i1, 500, tlong, 500 ,t1,"the quick fox jumped way over the lazy dog"
+    );
+    indexr(id,6, i1, -600, tlong, 600 ,t1,"humpty dumpy sat on a wall");
+    indexr(id,7, i1, 123, tlong, 123 ,t1,"humpty dumpy had a great fall");
+    indexr(id,8, i1, 876, tlong, 876,t1,"all the kings horses and all the kings men");
+    indexr(id,9, i1, 7, tlong, 7,t1,"couldn't put humpty together again");
+    indexr(id,10, i1, 4321, tlong, 4321,t1,"this too shall pass");
+    indexr(id,11, i1, -987, tlong, 987,t1,"An eye for eye only ends up making the whole world blind.");
+    indexr(id,12, i1, 379, tlong, 379,t1,"Great works are performed, not by strength, but by perseverance.");
+    indexr(id,13, i1, 232, tlong, 232,t1,"no eggs on wall, lesson learned", oddField, "odd man out");
+
+    indexr(id, 14, "SubjectTerms_mfacet", new String[]  {"mathematical models", "mathematical analysis"});
+    indexr(id, 15, "SubjectTerms_mfacet", new String[]  {"test 1", "test 2", "test3"});
+    indexr(id, 16, "SubjectTerms_mfacet", new String[]  {"test 1", "test 2", "test3"});
+    String[] vals = new String[100];
+    for (int i=0; i<100; i++) {
+      vals[i] = "test " + i;
+    }
+    indexr(id, 17, "SubjectTerms_mfacet", vals);
+
+    for (int i=100; i<150; i++) {
+      indexr(id, i);      
+    }
+
+    commit();
+
+    handle.clear();
+    handle.put("QTime", SKIPVAL);
+    handle.put("timestamp", SKIPVAL);
+
+    // random value sort
+    for (String f : fieldNames) {
+      query("q","*:*", "sort",f+" desc");
+      query("q","*:*", "sort",f+" asc");
+    }
+
+    // these queries should be exactly ordered and scores should exactly match
+    query("q","*:*", "sort",i1+" desc");
+    query("q","*:*", "sort",i1+" asc");
+    query("q","*:*", "sort",i1+" desc", "fl","*,score");
+    query("q","*:*", "sort",tlong+" asc", "fl","score");  // test legacy behavior - "score"=="*,score"
+    query("q","*:*", "sort",tlong+" desc");
+    handle.put("maxScore", SKIPVAL);
+    query("q","{!func}"+i1);// does not expect maxScore. So if it comes ,ignore it. JavaBinCodec.writeSolrDocumentList()
+    //is agnostic of request params.
+    handle.remove("maxScore");
+    query("q","{!func}"+i1, "fl","*,score");  // even scores should match exactly here
+
+    handle.put("highlighting", UNORDERED);
+    handle.put("response", UNORDERED);
+
+    handle.put("maxScore", SKIPVAL);
+    query("q","quick");
+    query("q","all","fl","id","start","0");
+    query("q","all","fl","foofoofoo","start","0");  // no fields in returned docs
+    query("q","all","fl","id","start","100");
+
+    handle.put("score", SKIPVAL);
+    query("q","quick","fl","*,score");
+    query("q","all","fl","*,score","start","1");
+    query("q","all","fl","*,score","start","100");
+
+    query("q","now their fox sat had put","fl","*,score",
+            "hl","true","hl.fl",t1);
+
+    query("q","now their fox sat had put","fl","foofoofoo",
+            "hl","true","hl.fl",t1);
+
+    query("q","matchesnothing","fl","*,score");  
+
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1);
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1, "facet.limit",-1, "facet.sort","count");
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1, "facet.limit",-1, "facet.sort","count", "facet.mincount",2);
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1, "facet.limit",-1, "facet.sort","index");
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1, "facet.limit",-1, "facet.sort","index", "facet.mincount",2);
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1,"facet.limit",1);
+    query("q","*:*", "rows",100, "facet","true", "facet.query","quick", "facet.query","all", "facet.query","*:*");
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1, "facet.offset",1);
+    query("q","*:*", "rows",100, "facet","true", "facet.field",t1, "facet.mincount",2);
+
+    // test faceting multiple things at once
+    query("q","*:*", "rows",100, "facet","true", "facet.query","quick", "facet.query","all", "facet.query","*:*"
+    ,"facet.field",t1);
+
+    // test filter tagging, facet exclusion, and naming (multi-select facet support)
+    query("q","*:*", "rows",100, "facet","true", "facet.query","{!key=myquick}quick", "facet.query","{!key=myall ex=a}all", "facet.query","*:*"
+    ,"facet.field","{!key=mykey ex=a}"+t1
+    ,"facet.field","{!key=other ex=b}"+t1
+    ,"facet.field","{!key=again ex=a,b}"+t1
+    ,"facet.field",t1
+    ,"fq","{!tag=a}id:[1 TO 7]", "fq","{!tag=b}id:[3 TO 9]"
+    );
+    query("q", "*:*", "facet", "true", "facet.field", "{!ex=t1}SubjectTerms_mfacet", "fq", "{!tag=t1}SubjectTerms_mfacet:(test 1)", "facet.limit", "10", "facet.mincount", "1");
+
+    // test field that is valid in schema but missing in all shards
+    query("q","*:*", "rows",100, "facet","true", "facet.field",missingField, "facet.mincount",2);
+    // test field that is valid in schema and missing in some shards
+    query("q","*:*", "rows",100, "facet","true", "facet.field",oddField, "facet.mincount",2);
+
+    query("q","*:*", "sort",i1+" desc", "stats", "true", "stats.field", i1);
+
+    /*** TODO: the failure may come back in "exception"
+    try {
+      // test error produced for field that is invalid for schema
+      query("q","*:*", "rows",100, "facet","true", "facet.field",invalidField, "facet.mincount",2);
+      TestCase.fail("SolrServerException expected for invalid field that is not in schema");
+    } catch (SolrServerException ex) {
+      // expected
+    }
+    ***/
+
+    // Try to get better coverage for refinement queries by turning off over requesting.
+    // This makes it much more likely that we may not get the top facet values and hence
+    // we turn of that checking.
+    handle.put("facet_fields", SKIPVAL);    
+    query("q","*:*", "rows",0, "facet","true", "facet.field",t1,"facet.limit",5, "facet.shard.limit",5);
+    // check a complex key name
+    query("q","*:*", "rows",0, "facet","true", "facet.field","{!key='a b/c \\' \\} foo'}"+t1,"facet.limit",5, "facet.shard.limit",5);
+    handle.remove("facet_fields");
+
+
+    // index the same document to two servers and make sure things
+    // don't blow up.
+    if (clients.size()>=2) {
+      index(id,100, i1, 107 ,t1,"oh no, a duplicate!");
+      for (int i=0; i<clients.size(); i++) {
+        index_specific(i, id,100, i1, 107 ,t1,"oh no, a duplicate!");
+      }
+      commit();
+      query("q","duplicate", "hl","true", "hl.fl", t1);
+      query("q","fox duplicate horses", "hl","true", "hl.fl", t1);
+      query("q","*:*", "rows",100);
+    }
+
+    // test debugging
+    handle.put("explain", UNORDERED);
+    handle.put("debug", UNORDERED);
+    handle.put("time", SKIPVAL);
+    query("q","now their fox sat had put","fl","*,score",CommonParams.DEBUG_QUERY, "true");
+    query("q", "id:[1 TO 5]", CommonParams.DEBUG_QUERY, "true");
+    query("q", "id:[1 TO 5]", CommonParams.DEBUG, CommonParams.TIMING);
+    query("q", "id:[1 TO 5]", CommonParams.DEBUG, CommonParams.RESULTS);
+    query("q", "id:[1 TO 5]", CommonParams.DEBUG, CommonParams.QUERY);
+
+    // TODO: This test currently fails because debug info is obtained only
+    // on shards with matches.
+    // query("q","matchesnothing","fl","*,score", "debugQuery", "true");
+
+    // Thread.sleep(10000000000L);
+    if (DEBUG) {
+      super.printLayout();
+    }
+  }
+
+  volatile CloudSolrServer solrj;
+
+  @Override
+  protected QueryResponse queryServer(ModifiableSolrParams params) throws SolrServerException {
+
+    if (r.nextBoolean())
+      return super.queryServer(params);
+
+    // use the distributed solrj client
+    if (solrj == null) {
+      synchronized(this) {
+        try {
+          CloudSolrServer server = new CloudSolrServer(zkServer.getZkAddress());
+          server.setDefaultCollection(DEFAULT_COLLECTION);
+          solrj = server;
+        } catch (MalformedURLException e) {
+          throw new RuntimeException(e);
+        }
+      }
+    }
+
+    if (r.nextBoolean())
+      params.set("collection",DEFAULT_COLLECTION);
+
+    QueryResponse rsp = solrj.query(params);
+    return rsp;
+  }
+  
+  @Override
+  public void tearDown() throws Exception {
+    super.tearDown();
+    System.clearProperty("CLOUD_UPDATE_DELAY");
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicZkTest.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicZkTest.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicZkTest.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/BasicZkTest.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,129 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import org.apache.lucene.index.LogMergePolicy;
+import org.apache.solr.core.SolrCore;
+import org.apache.solr.update.SolrIndexWriter;
+
+import org.junit.BeforeClass;
+import org.junit.Test;
+import static org.junit.Assert.*;
+
+/**
+ *
+ */
+public class BasicZkTest extends AbstractZkTestCase {
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    initCore("solrconfig.xml", "schema.xml");
+  }
+  
+  @Test
+  public void testBasic() throws Exception {
+    // test using ZooKeeper
+    assertTrue("Not using ZooKeeper", h.getCoreContainer().isZooKeeperAware());
+    
+    ZkController zkController = h.getCoreContainer().getZkController();
+    
+    // test merge factor picked up
+    SolrCore core = h.getCore();
+    SolrIndexWriter writer = new SolrIndexWriter("testWriter", core
+        .getNewIndexDir(), core.getDirectoryFactory(), false, core.getSchema(),
+        core.getSolrConfig().mainIndexConfig, core.getDeletionPolicy());
+    assertEquals("Mergefactor was not picked up", ((LogMergePolicy)writer.getConfig().getMergePolicy()).getMergeFactor(), 8);
+    writer.close();
+    
+    lrf.args.put("version", "2.0");
+    assertQ("test query on empty index", req("qlkciyopsbgzyvkylsjhchghjrdf"),
+        "//result[@numFound='0']");
+
+    // test escaping of ";"
+    assertU("deleting 42 for no reason at all", delI("42"));
+    assertU("adding doc#42", adoc("id", "42", "val_s", "aa;bb"));
+    assertU("does commit work?", commit());
+
+    assertQ("backslash escaping semicolon", req("id:42 AND val_s:aa\\;bb"),
+        "//*[@numFound='1']", "//int[@name='id'][.='42']");
+
+    assertQ("quote escaping semicolon", req("id:42 AND val_s:\"aa;bb\""),
+        "//*[@numFound='1']", "//int[@name='id'][.='42']");
+
+    assertQ("no escaping semicolon", req("id:42 AND val_s:aa"),
+        "//*[@numFound='0']");
+
+    assertU(delI("42"));
+    assertU(commit());
+    assertQ(req("id:42"), "//*[@numFound='0']");
+
+    // test allowDups default of false
+
+    assertU(adoc("id", "42", "val_s", "AAA"));
+    assertU(adoc("id", "42", "val_s", "BBB"));
+    assertU(commit());
+    assertQ(req("id:42"), "//*[@numFound='1']", "//str[.='BBB']");
+    assertU(adoc("id", "42", "val_s", "CCC"));
+    assertU(adoc("id", "42", "val_s", "DDD"));
+    assertU(commit());
+    assertQ(req("id:42"), "//*[@numFound='1']", "//str[.='DDD']");
+
+    // test deletes
+    String[] adds = new String[] { add(doc("id", "101"), "allowDups", "false"),
+        add(doc("id", "101"), "allowDups", "false"),
+        add(doc("id", "105"), "allowDups", "true"),
+        add(doc("id", "102"), "allowDups", "false"),
+        add(doc("id", "103"), "allowDups", "true"),
+        add(doc("id", "101"), "allowDups", "false"), };
+    for (String a : adds) {
+      assertU(a, a);
+    }
+    assertU(commit());
+    
+    zkServer.shutdown();
+    
+    Thread.sleep(300);
+    
+    // try a reconnect from disconnect
+    
+    zkServer = new ZkTestServer(zkDir);
+    zkServer.run();
+    
+    // ensure zk still thinks node is up
+    assertTrue(zkController.getCloudState().liveNodesContain(zkController.getNodeName()));
+    
+    // test maxint
+    assertQ(req("q", "id:[100 TO 110]", "rows", "2147483647"),
+        "//*[@numFound='4']");
+
+    // test big limit
+    assertQ(req("q", "id:[100 TO 111]", "rows", "1147483647"),
+        "//*[@numFound='4']");
+
+    assertQ(req("id:[100 TO 110]"), "//*[@numFound='4']");
+    assertU(delI("102"));
+    assertU(commit());
+    assertQ(req("id:[100 TO 110]"), "//*[@numFound='3']");
+    assertU(delI("105"));
+    assertU(commit());
+    assertQ(req("id:[100 TO 110]"), "//*[@numFound='2']");
+    assertU(delQ("id:[100 TO 110]"));
+    assertU(commit());
+    assertQ(req("id:[100 TO 110]"), "//*[@numFound='0']");
+
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/CloudStateUpdateTest.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/CloudStateUpdateTest.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/CloudStateUpdateTest.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/CloudStateUpdateTest.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,248 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.io.File;
+import java.util.Map;
+import java.util.Set;
+
+import org.apache.solr.SolrTestCaseJ4;
+import org.apache.solr.common.cloud.CloudState;
+import org.apache.solr.common.cloud.Slice;
+import org.apache.solr.common.cloud.SolrZkClient;
+import org.apache.solr.common.cloud.ZkNodeProps;
+import org.apache.solr.core.CoreContainer;
+import org.apache.solr.core.CoreDescriptor;
+import org.apache.solr.core.SolrConfig;
+import org.apache.solr.core.SolrCore;
+import org.apache.solr.core.CoreContainer.Initializer;
+import org.apache.zookeeper.CreateMode;
+import org.junit.BeforeClass;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import org.junit.Test;
+import static org.junit.Assert.*;
+
+/**
+ * TODO: look at hostPort used below
+ */
+public class CloudStateUpdateTest extends SolrTestCaseJ4 {
+  protected static Logger log = LoggerFactory
+      .getLogger(AbstractZkTestCase.class);
+
+  private static final boolean VERBOSE = false;
+
+  protected ZkTestServer zkServer;
+
+  protected String zkDir;
+
+  private CoreContainer container1;
+
+  private CoreContainer container2;
+
+  private CoreContainer container3;
+
+  private File dataDir1;
+
+  private File dataDir2;
+
+  private File dataDir3;
+  
+  private File dataDir4;
+
+  private Initializer init2;
+  
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    initCore();
+  }
+
+  @Override
+  public void setUp() throws Exception {
+    super.setUp();
+    
+    System.setProperty("zkClientTimeout", "3000");
+    
+    zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    zkServer = new ZkTestServer(zkDir);
+    zkServer.run();
+    System.setProperty("zkHost", zkServer.getZkAddress());
+    AbstractZkTestCase.buildZooKeeper(zkServer.getZkHost(), zkServer
+        .getZkAddress(), "solrconfig.xml", "schema.xml");
+    
+    log.info("####SETUP_START " + getName());
+    dataDir1 = new File(dataDir + File.separator + "data1");
+    dataDir1.mkdirs();
+    
+    dataDir2 = new File(dataDir + File.separator + "data2");
+    dataDir2.mkdirs();
+    
+    dataDir3 = new File(dataDir + File.separator + "data3");
+    dataDir3.mkdirs();
+    
+    dataDir4 = new File(dataDir + File.separator + "data3");
+    dataDir4.mkdirs();
+    
+    // set some system properties for use by tests
+    System.setProperty("solr.test.sys.prop1", "propone");
+    System.setProperty("solr.test.sys.prop2", "proptwo");
+    
+    System.setProperty("hostPort", "1661");
+    CoreContainer.Initializer init1 = new CoreContainer.Initializer() {
+      {
+        this.dataDir = CloudStateUpdateTest.this.dataDir1.getAbsolutePath();
+      }
+    };
+    
+    container1 = init1.initialize();
+    System.clearProperty("hostPort");
+    
+    System.setProperty("hostPort", "1662");
+    init2 = new CoreContainer.Initializer() {
+      {
+        this.dataDir = CloudStateUpdateTest.this.dataDir2.getAbsolutePath();
+      }
+    };
+    
+    container2 = init2.initialize();
+    System.clearProperty("hostPort");
+    
+    System.setProperty("hostPort", "1663");
+    CoreContainer.Initializer init3 = new CoreContainer.Initializer() {
+      {
+        this.dataDir = CloudStateUpdateTest.this.dataDir3.getAbsolutePath();
+      }
+    };
+    container3 = init3.initialize();
+    System.clearProperty("hostPort");
+    
+    log.info("####SETUP_END " + getName());
+    
+  }
+
+  @Test
+  public void testCoreRegistration() throws Exception {
+    System.setProperty("CLOUD_UPDATE_DELAY", "1");
+    
+    ZkNodeProps props2 = new ZkNodeProps();
+    props2.put("configName", "conf1");
+    
+    SolrZkClient zkClient = new SolrZkClient(zkServer.getZkAddress(), AbstractZkTestCase.TIMEOUT);
+    zkClient.makePath("/collections/testcore", props2.store(), CreateMode.PERSISTENT);
+    zkClient.makePath("/collections/testcore/shards", CreateMode.PERSISTENT);
+    zkClient.close();
+    
+    CoreDescriptor dcore = new CoreDescriptor(container1, "testcore",
+        "testcore");
+    
+    dcore.setDataDir(dataDir4.getAbsolutePath());
+
+    SolrCore core = container1.create(dcore);
+    container1.register(core, false);
+    
+    // slight pause - TODO: takes an oddly long amount of time to schedule tasks
+    // with almost no delay ...
+    Thread.sleep(5000);
+
+    ZkController zkController2 = container2.getZkController();
+
+    String host = zkController2.getHostName();
+
+    CloudState cloudState2 = zkController2.getCloudState();
+    Map<String,Slice> slices = cloudState2.getSlices("testcore");
+
+    assertNotNull(slices);
+    assertTrue(slices.containsKey(host + ":1661_solr_testcore"));
+
+    Slice slice = slices.get(host + ":1661_solr_testcore");
+    assertEquals(host + ":1661_solr_testcore", slice.getName());
+
+    Map<String,ZkNodeProps> shards = slice.getShards();
+
+    assertEquals(1, shards.size());
+
+    ZkNodeProps zkProps = shards.get(host + ":1661_solr_testcore");
+
+    assertNotNull(zkProps);
+
+    assertEquals(host + ":1661_solr", zkProps.get("node_name"));
+
+    assertEquals("http://" + host + ":1661/solr/testcore", zkProps.get("url"));
+
+    Set<String> liveNodes = cloudState2.getLiveNodes();
+    assertNotNull(liveNodes);
+    assertEquals(3, liveNodes.size());
+
+    container3.shutdown();
+
+    // slight pause for watch to trigger
+    for(int i = 0; i < 4; i++) {
+      if(zkController2.getCloudState().getLiveNodes().size() == 2) {
+        break;
+      }
+      Thread.sleep(50);
+    }
+
+    assertEquals(2, zkController2.getCloudState().getLiveNodes().size());
+
+    // quickly kill / start client
+
+    container2.getZkController().getZkClient().getSolrZooKeeper().getConnection()
+        .disconnect();
+    container2.shutdown();
+
+    container2 = init2.initialize();
+
+    Thread.sleep(2000);
+    
+    if (!container1.getZkController().getCloudState().liveNodesContain(
+        container2.getZkController().getNodeName())) {
+      // pause some more
+      Thread.sleep(5000);
+    }
+
+    assertTrue(container1.getZkController().getCloudState().liveNodesContain(
+        container2.getZkController().getNodeName()));
+    
+  }
+
+  public void tearDown() throws Exception {
+    if (VERBOSE) {
+      printLayout(zkServer.getZkHost());
+    }
+    container1.shutdown();
+    container2.shutdown();
+    container3.shutdown();
+    zkServer.shutdown();
+    super.tearDown();
+    System.clearProperty("zkClientTimeout");
+    System.clearProperty("zkHost");
+    System.clearProperty("hostPort");
+    System.clearProperty("CLOUD_UPDATE_DELAY");
+    SolrConfig.severeErrors.clear();
+  }
+
+  private void printLayout(String zkHost) throws Exception {
+    SolrZkClient zkClient = new SolrZkClient(
+        zkHost, AbstractZkTestCase.TIMEOUT);
+    zkClient.printLayoutToStdOut();
+    zkClient.close();
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkControllerTest.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkControllerTest.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkControllerTest.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkControllerTest.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,225 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.io.File;
+import java.io.IOException;
+import java.util.Map;
+
+import org.apache.solr.SolrTestCaseJ4;
+import org.apache.solr.common.cloud.CloudState;
+import org.apache.solr.common.cloud.Slice;
+import org.apache.solr.common.cloud.SolrZkClient;
+import org.apache.solr.common.cloud.ZkNodeProps;
+import org.apache.solr.common.cloud.ZkStateReader;
+import org.apache.solr.core.SolrConfig;
+import org.apache.zookeeper.CreateMode;
+import org.apache.zookeeper.KeeperException;
+
+import org.junit.BeforeClass;
+import org.junit.Test;
+import static org.junit.Assert.*;
+
+public class ZkControllerTest extends SolrTestCaseJ4 {
+
+  private static final String TEST_NODE_NAME = "test_node_name";
+
+  private static final String URL3 = "http://localhost:3133/solr/core1";
+
+  private static final String URL2 = "http://localhost:3123/solr/core1";
+
+  private static final String SHARD3 = "localhost:3123_solr_core3";
+
+  private static final String SHARD2 = "localhost:3123_solr_core2";
+
+  private static final String SHARD1 = "localhost:3123_solr_core1";
+
+  private static final String COLLECTION_NAME = "collection1";
+
+  static final int TIMEOUT = 10000;
+
+  private static final String URL1 = "http://localhost:3133/solr/core0";
+
+  private static final boolean DEBUG = false;
+  
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    initCore();
+  }
+
+  @Test
+  public void testReadShards() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    ZkTestServer server = null;
+    SolrZkClient zkClient = null;
+    ZkController zkController = null;
+    try {
+      server = new ZkTestServer(zkDir);
+      server.run();
+
+      AbstractZkTestCase.makeSolrZkNode(server.getZkHost());
+
+      zkClient = new SolrZkClient(server.getZkAddress(), TIMEOUT);
+      String shardsPath = "/collections/collection1/shards/shardid1";
+      zkClient.makePath(shardsPath);
+
+      addShardToZk(zkClient, shardsPath, SHARD1, URL1);
+      addShardToZk(zkClient, shardsPath, SHARD2, URL2);
+      addShardToZk(zkClient, shardsPath, SHARD3, URL3);
+
+      if (DEBUG) {
+        zkClient.printLayoutToStdOut();
+      }
+
+      zkController = new ZkController(server.getZkAddress(),
+          TIMEOUT, 1000, "localhost", "8983", "solr");
+ 
+      zkController.getZkStateReader().updateCloudState(true);
+      CloudState cloudInfo = zkController.getCloudState();
+      Map<String,Slice> slices = cloudInfo.getSlices("collection1");
+      assertNotNull(slices);
+
+      for (Slice slice : slices.values()) {
+        Map<String,ZkNodeProps> shards = slice.getShards();
+        if (DEBUG) {
+          for (String shardName : shards.keySet()) {
+            ZkNodeProps props = shards.get(shardName);
+            System.out.println("shard:" + shardName);
+            System.out.println("props:" + props.toString());
+          }
+        }
+        assertNotNull(shards.get(SHARD1));
+        assertNotNull(shards.get(SHARD2));
+        assertNotNull(shards.get(SHARD3));
+
+        ZkNodeProps props = shards.get(SHARD1);
+        assertEquals(URL1, props.get(ZkStateReader.URL_PROP));
+        assertEquals(TEST_NODE_NAME, props.get(ZkStateReader.NODE_NAME));
+
+        props = shards.get(SHARD2);
+        assertEquals(URL2, props.get(ZkStateReader.URL_PROP));
+        assertEquals(TEST_NODE_NAME, props.get(ZkStateReader.NODE_NAME));
+
+        props = shards.get(SHARD3);
+        assertEquals(URL3, props.get(ZkStateReader.URL_PROP));
+        assertEquals(TEST_NODE_NAME, props.get(ZkStateReader.NODE_NAME));
+
+      }
+
+    } finally {
+      if (zkClient != null) {
+        zkClient.close();
+      }
+      if (zkController != null) {
+        zkController.close();
+      }
+      if (server != null) {
+        server.shutdown();
+      }
+    }
+  }
+
+  @Test
+  public void testReadConfigName() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+
+    ZkTestServer server = new ZkTestServer(zkDir);
+    try {
+      server.run();
+
+      AbstractZkTestCase.makeSolrZkNode(server.getZkHost());
+
+      SolrZkClient zkClient = new SolrZkClient(server.getZkAddress(), TIMEOUT);
+      String actualConfigName = "firstConfig";
+
+      zkClient.makePath(ZkController.CONFIGS_ZKNODE + "/" + actualConfigName);
+      
+      ZkNodeProps props = new ZkNodeProps();
+      props.put("configName", actualConfigName);
+      zkClient.makePath(ZkStateReader.COLLECTIONS_ZKNODE + "/" + COLLECTION_NAME , props.store(), CreateMode.PERSISTENT);
+
+      if (DEBUG) {
+        zkClient.printLayoutToStdOut();
+      }
+      zkClient.close();
+      ZkController zkController = new ZkController(server.getZkAddress(), TIMEOUT, 1000,
+          "localhost", "8983", "/solr");
+      try {
+        String configName = zkController.readConfigName(COLLECTION_NAME);
+        assertEquals(configName, actualConfigName);
+      } finally {
+        zkController.close();
+      }
+    } finally {
+
+      server.shutdown();
+    }
+
+  }
+
+  @Test
+  public void testUploadToCloud() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+
+    ZkTestServer server = new ZkTestServer(zkDir);
+    ZkController zkController = null;
+    try {
+      server.run();
+
+      AbstractZkTestCase.makeSolrZkNode(server.getZkHost());
+
+      zkController = new ZkController(server.getZkAddress(),
+          TIMEOUT, 1000, "localhost", "8983", "/solr");
+
+      zkController.uploadToZK(new File("solr/conf"),
+          ZkController.CONFIGS_ZKNODE + "/config1");
+
+      if (DEBUG) {
+        zkController.printLayoutToStdOut();
+      }
+
+    } finally {
+      if (zkController != null) {
+        zkController.close();
+      }
+      server.shutdown();
+    }
+
+  }
+
+  private void addShardToZk(SolrZkClient zkClient, String shardsPath,
+      String zkNodeName, String url) throws IOException,
+      KeeperException, InterruptedException {
+
+    ZkNodeProps props = new ZkNodeProps();
+    props.put(ZkStateReader.URL_PROP, url);
+    props.put(ZkStateReader.NODE_NAME, TEST_NODE_NAME);
+    byte[] bytes = props.store();
+
+    zkClient
+        .create(shardsPath + "/" + zkNodeName, bytes, CreateMode.PERSISTENT);
+  }
+  
+  public void tearDown() throws Exception {
+    SolrConfig.severeErrors.clear();
+    super.tearDown();
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkNodePropsTest.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkNodePropsTest.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkNodePropsTest.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkNodePropsTest.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,51 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.io.IOException;
+
+import org.apache.solr.SolrTestCaseJ4;
+import org.apache.solr.common.cloud.ZkNodeProps;
+
+import org.junit.Test;
+import static org.junit.Assert.*;
+
+
+public class ZkNodePropsTest extends SolrTestCaseJ4 {
+  @Test
+  public void testBasic() throws IOException {
+
+    ZkNodeProps props = new ZkNodeProps();
+    props.put("prop1", "value1");
+    props.put("prop2", "value2");
+    props.put("prop3", "value3");
+    props.put("prop4", "value4");
+    props.put("prop5", "value5");
+    props.put("prop6", "value6");
+    byte[] bytes = props.store();
+    
+    ZkNodeProps props2 = new ZkNodeProps();
+    props2.load(bytes);
+    assertEquals("value1", props2.get("prop1"));
+    assertEquals("value2", props2.get("prop2"));
+    assertEquals("value3", props2.get("prop3"));
+    assertEquals("value4", props2.get("prop4"));
+    assertEquals("value5", props2.get("prop5"));
+    assertEquals("value6", props2.get("prop6"));
+  }
+}

Added: lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkSolrClientTest.java
URL: http://svn.apache.org/viewvc/lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkSolrClientTest.java?rev=1022188&view=auto
==============================================================================
--- lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkSolrClientTest.java (added)
+++ lucene/dev/trunk/solr/src/test/org/apache/solr/cloud/ZkSolrClientTest.java Wed Oct 13 17:01:13 2010
@@ -0,0 +1,240 @@
+package org.apache.solr.cloud;
+
+/**
+ * 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.
+ */
+
+import java.io.File;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import junit.framework.TestCase;
+
+import org.apache.solr.common.cloud.SolrZkClient;
+import org.apache.solr.core.SolrConfig;
+import org.apache.solr.util.AbstractSolrTestCase;
+import org.apache.zookeeper.KeeperException;
+import org.apache.zookeeper.WatchedEvent;
+import org.apache.zookeeper.Watcher;
+
+public class ZkSolrClientTest extends AbstractSolrTestCase {
+  private static final boolean DEBUG = false;
+
+  public void testConnect() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    ZkTestServer server = null;
+
+    server = new ZkTestServer(zkDir);
+    server.run();
+
+    SolrZkClient zkClient = new SolrZkClient(server.getZkAddress(), 100);
+
+    zkClient.close();
+    server.shutdown();
+  }
+
+  public void testMakeRootNode() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    ZkTestServer server = null;
+
+    server = new ZkTestServer(zkDir);
+    server.run();
+
+    AbstractZkTestCase.makeSolrZkNode(server.getZkHost());
+
+    SolrZkClient zkClient = new SolrZkClient(server.getZkHost(),
+        AbstractZkTestCase.TIMEOUT);
+
+    assertTrue(zkClient.exists("/solr"));
+
+    zkClient.close();
+    server.shutdown();
+  }
+
+  public void testReconnect() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    ZkTestServer server = null;
+    SolrZkClient zkClient = null;
+    try {
+      server = new ZkTestServer(zkDir);
+      server.run();
+
+      AbstractZkTestCase.makeSolrZkNode(server.getZkHost());
+
+      zkClient = new SolrZkClient(server.getZkAddress(), AbstractZkTestCase.TIMEOUT);
+      String shardsPath = "/collections/collection1/shards";
+      zkClient.makePath(shardsPath);
+
+      zkClient.makePath("collections/collection1");
+      int zkServerPort = server.getPort();
+      // this tests disconnect state
+      server.shutdown();
+
+      Thread.sleep(80);
+
+
+      try {
+        zkClient.makePath("collections/collection2");
+        TestCase.fail("Server should be down here");
+      } catch (KeeperException.ConnectionLossException e) {
+
+      }
+
+      // bring server back up
+      server = new ZkTestServer(zkDir, zkServerPort);
+      server.run();
+
+      // TODO: can we do better?
+      // wait for reconnect
+      Thread.sleep(600);
+
+      try {
+        zkClient.makePath("collections/collection3");
+      } catch (KeeperException.ConnectionLossException e) {
+        Thread.sleep(5000); // try again in a bit
+        zkClient.makePath("collections/collection3");
+      }
+
+      if (DEBUG) {
+        zkClient.printLayoutToStdOut();
+      }
+
+      assertNotNull(zkClient.exists("/collections/collection3", null));
+      assertNotNull(zkClient.exists("/collections/collection1", null));
+      
+      // simulate session expiration
+      
+      // one option
+      long sessionId = zkClient.getSolrZooKeeper().getSessionId();
+      server.expire(sessionId);
+      
+      // another option
+      //zkClient.getSolrZooKeeper().getConnection().disconnect();
+
+      // this tests expired state
+
+      Thread.sleep(1000); // pause for reconnect
+      
+      for (int i = 0; i < 8; i++) {
+        try {
+          zkClient.makePath("collections/collection4");
+          break;
+        } catch (KeeperException.SessionExpiredException e) {
+
+        } catch (KeeperException.ConnectionLossException e) {
+
+        }
+        Thread.sleep(1000 * i);
+      }
+
+      if (DEBUG) {
+        zkClient.printLayoutToStdOut();
+      }
+
+      assertNotNull("Node does not exist, but it should", zkClient.exists("/collections/collection4", null));
+
+    } finally {
+
+      if (zkClient != null) {
+        zkClient.close();
+      }
+      if (server != null) {
+        server.shutdown();
+      }
+    }
+  }
+
+  public void testWatchChildren() throws Exception {
+    String zkDir = dataDir.getAbsolutePath() + File.separator
+        + "zookeeper/server1/data";
+    
+    final AtomicInteger cnt = new AtomicInteger();
+    ZkTestServer server = new ZkTestServer(zkDir);
+    server.run();
+    Thread.sleep(400);
+    AbstractZkTestCase.makeSolrZkNode(server.getZkHost());
+    final SolrZkClient zkClient = new SolrZkClient(server.getZkAddress(), AbstractZkTestCase.TIMEOUT);
+    try {
+      zkClient.makePath("/collections");
+
+      zkClient.getChildren("/collections", new Watcher() {
+
+        public void process(WatchedEvent event) {
+          if (DEBUG) {
+            System.out.println("children changed");
+          }
+          cnt.incrementAndGet();
+          // remake watch
+          try {
+            zkClient.getChildren("/collections", this);
+          } catch (KeeperException e) {
+            throw new RuntimeException(e);
+          } catch (InterruptedException e) {
+            throw new RuntimeException(e);
+          }
+        }
+      });
+
+      zkClient.makePath("/collections/collection99/shards");
+
+      zkClient.makePath("collections/collection99/config=collection1");
+
+      zkClient.makePath("collections/collection99/config=collection3");
+      
+      zkClient.makePath("/collections/collection97/shards");
+
+      if (DEBUG) {
+        zkClient.printLayoutToStdOut();
+      }
+      
+      // pause for the watches to fire
+      Thread.sleep(700);
+      
+      if (cnt.intValue() < 2) {
+        Thread.sleep(4000); // wait a bit more
+      }
+      
+      assertEquals(2, cnt.intValue());
+
+    } finally {
+
+      if (zkClient != null) {
+        zkClient.close();
+      }
+      if (server != null) {
+        server.shutdown();
+      }
+    }
+  }
+
+  @Override
+  public String getSchemaFile() {
+    return null;
+  }
+
+  @Override
+  public String getSolrConfigFile() {
+    return null;
+  }
+  
+  public void tearDown() throws Exception {
+    SolrConfig.severeErrors.clear();
+    super.tearDown();
+  }
+  
+}