You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@accumulo.apache.org by el...@apache.org on 2014/03/20 22:49:18 UTC

[01/10] git commit: ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()

Repository: accumulo
Updated Branches:
  refs/heads/1.4.5-SNAPSHOT bbd6aeb33 -> 0753a754b
  refs/heads/1.5.2-SNAPSHOT 4b7a011ca -> 63d5e55a0
  refs/heads/1.6.0-SNAPSHOT aefa52e2a -> 44b13c12e
  refs/heads/master b644edade -> 394fe061f


ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/0753a754
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/0753a754
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/0753a754

Branch: refs/heads/1.4.5-SNAPSHOT
Commit: 0753a754b0a1751ab7f2bb4ddd922aa9dd492252
Parents: bbd6aeb
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:35:04 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:35:04 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java  | 17 +++++++++++++----
 1 file changed, 13 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/0753a754/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --git a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index cbed375..848e14f 100644
--- a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@ -362,22 +362,31 @@ public class MiniAccumuloCluster {
    */
   
   public void stop() throws IOException, InterruptedException {
-    if (zooKeeperProcess != null)
+    if (zooKeeperProcess != null) {
       zooKeeperProcess.destroy();
-    if (loggerProcess != null)
+      zooKeeperProcess.waitFor();
+    }
+    if (loggerProcess != null) {
       loggerProcess.destroy();
-    if (masterProcess != null)
+      loggerProcess.waitFor();
+    }
+    if (masterProcess != null) {
       masterProcess.destroy();
+      masterProcess.waitFor();
+    }
     if (tabletServerProcesses != null) {
       for (Process tserver : tabletServerProcesses) {
         tserver.destroy();
+        tserver.waitFor();
       }
     }
     
     for (LogWriter lw : logWriters)
       lw.flush();
 
-    if (gcProcess != null)
+    if (gcProcess != null) {
       gcProcess.destroy();
+      gcProcess.waitFor();
+    }
   }
 }


[08/10] git commit: Merge branch '1.5.2-SNAPSHOT' into 1.6.0-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.5.2-SNAPSHOT' into 1.6.0-SNAPSHOT

Conflicts:
	minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/44b13c12
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/44b13c12
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/44b13c12

Branch: refs/heads/1.6.0-SNAPSHOT
Commit: 44b13c12e5555b9da6165affe038f825a6b17fa2
Parents: aefa52e 63d5e55
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 17:48:51 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 17:48:51 2014 -0400

----------------------------------------------------------------------
 .../minicluster/impl/MiniAccumuloClusterImpl.java       | 12 +++++++++++-
 1 file changed, 11 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/44b13c12/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
----------------------------------------------------------------------
diff --cc minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
index 8a9736d,0000000..009988e
mode 100644,000000..100644
--- a/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
+++ b/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
@@@ -1,649 -1,0 +1,659 @@@
 +/*
 + * Licensed to the Apache Software Foundation (ASF) under one or more
 + * contributor license agreements.  See the NOTICE file distributed with
 + * this work for additional information regarding copyright ownership.
 + * The ASF licenses this file to You under the Apache License, Version 2.0
 + * (the "License"); you may not use this file except in compliance with
 + * the License.  You may obtain a copy of the License at
 + *
 + *     http://www.apache.org/licenses/LICENSE-2.0
 + *
 + * Unless required by applicable law or agreed to in writing, software
 + * distributed under the License is distributed on an "AS IS" BASIS,
 + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
 + * See the License for the specific language governing permissions and
 + * limitations under the License.
 + */
 +package org.apache.accumulo.minicluster.impl;
 +
 +import java.io.BufferedReader;
 +import java.io.BufferedWriter;
 +import java.io.File;
 +import java.io.FileFilter;
 +import java.io.FileWriter;
 +import java.io.IOException;
 +import java.io.InputStream;
 +import java.io.InputStreamReader;
 +import java.net.InetSocketAddress;
 +import java.net.Socket;
 +import java.net.URI;
 +import java.net.URISyntaxException;
 +import java.net.URL;
 +import java.net.URLClassLoader;
 +import java.util.ArrayList;
 +import java.util.Arrays;
 +import java.util.Collection;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.HashSet;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Properties;
 +import java.util.Set;
 +
 +import org.apache.accumulo.core.client.AccumuloException;
 +import org.apache.accumulo.core.client.AccumuloSecurityException;
 +import org.apache.accumulo.core.client.ClientConfiguration;
 +import org.apache.accumulo.core.client.Connector;
 +import org.apache.accumulo.core.client.Instance;
 +import org.apache.accumulo.core.client.ZooKeeperInstance;
 +import org.apache.accumulo.core.client.security.tokens.PasswordToken;
 +import org.apache.accumulo.core.conf.Property;
 +import org.apache.accumulo.core.master.thrift.MasterGoalState;
 +import org.apache.accumulo.core.util.Daemon;
 +import org.apache.accumulo.core.util.Pair;
 +import org.apache.accumulo.core.util.StringUtil;
 +import org.apache.accumulo.core.util.UtilWaitThread;
 +import org.apache.accumulo.gc.SimpleGarbageCollector;
 +import org.apache.accumulo.master.Master;
 +import org.apache.accumulo.master.state.SetGoalState;
 +import org.apache.accumulo.minicluster.ServerType;
 +import org.apache.accumulo.server.init.Initialize;
 +import org.apache.accumulo.server.util.PortUtils;
 +import org.apache.accumulo.server.util.time.SimpleTimer;
 +import org.apache.accumulo.start.Main;
 +import org.apache.accumulo.start.classloader.vfs.MiniDFSUtil;
 +import org.apache.accumulo.tserver.TabletServer;
 +import org.apache.commons.configuration.MapConfiguration;
 +import org.apache.commons.vfs2.FileObject;
 +import org.apache.commons.vfs2.impl.VFSClassLoader;
 +import org.apache.hadoop.conf.Configuration;
 +import org.apache.hadoop.fs.FileSystem;
 +import org.apache.hadoop.hdfs.DFSConfigKeys;
 +import org.apache.hadoop.hdfs.MiniDFSCluster;
 +import org.apache.zookeeper.server.ZooKeeperServerMain;
 +
 +import com.google.common.base.Predicate;
 +import com.google.common.collect.Maps;
 +
 +/**
 + * A utility class that will create Zookeeper and Accumulo processes that write all of their data to a single local directory. This class makes it easy to test
 + * code against a real Accumulo instance. Its much more accurate for testing than {@link org.apache.accumulo.core.client.mock.MockAccumulo}, but much slower.
 + * 
 + * @since 1.5.0
 + */
 +public class MiniAccumuloClusterImpl {
 +
 +  public static class LogWriter extends Daemon {
 +    private BufferedReader in;
 +    private BufferedWriter out;
 +
 +    public LogWriter(InputStream stream, File logFile) throws IOException {
 +      this.in = new BufferedReader(new InputStreamReader(stream));
 +      out = new BufferedWriter(new FileWriter(logFile));
 +
 +      SimpleTimer.getInstance().schedule(new Runnable() {
 +        @Override
 +        public void run() {
 +          try {
 +            flush();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      }, 1000, 1000);
 +    }
 +
 +    public synchronized void flush() throws IOException {
 +      if (out != null)
 +        out.flush();
 +    }
 +
 +    @Override
 +    public void run() {
 +      String line;
 +
 +      try {
 +        while ((line = in.readLine()) != null) {
 +          out.append(line);
 +          out.append("\n");
 +        }
 +
 +        synchronized (this) {
 +          out.close();
 +          out = null;
 +          in.close();
 +        }
 +
 +      } catch (IOException e) {}
 +    }
 +  }
 +
 +  private static final long ZOOKEEPER_STARTUP_WAIT = 20*1000;
 +
 +  private boolean initialized = false;
 +  private Process zooKeeperProcess = null;
 +  private Process masterProcess = null;
 +  private Process gcProcess = null;
 +  private List<Process> tabletServerProcesses = Collections.synchronizedList(new ArrayList<Process>());
 +
 +  private Set<Pair<ServerType,Integer>> debugPorts = new HashSet<Pair<ServerType,Integer>>();
 +
 +  private File zooCfgFile;
 +  private String dfsUri;
 +
 +  public List<LogWriter> getLogWriters() {
 +    return logWriters;
 +  }
 +
 +  private List<LogWriter> logWriters = new ArrayList<MiniAccumuloClusterImpl.LogWriter>();
 +
 +  private MiniAccumuloConfigImpl config;
 +  private MiniDFSCluster miniDFS = null;
 +  private List<Process> cleanup = new ArrayList<Process>();
 +
 +  public Process exec(Class<?> clazz, String... args) throws IOException {
 +    return exec(clazz, null, args);
 +  }
 +
 +  public Process exec(Class<?> clazz, List<String> jvmArgs, String... args) throws IOException {
 +    ArrayList<String> jvmArgs2 = new ArrayList<String>(1 + (jvmArgs == null ? 0 : jvmArgs.size()));
 +    jvmArgs2.add("-Xmx" + config.getDefaultMemory());
 +    if (jvmArgs != null)
 +      jvmArgs2.addAll(jvmArgs);
 +    Process proc = _exec(clazz, jvmArgs2, args);
 +    cleanup.add(proc);
 +    return proc;
 +  }
 +
 +  private boolean containsSiteFile(File f) {
 +    return f.isDirectory() && f.listFiles(new FileFilter() {
 +
 +      @Override
 +      public boolean accept(File pathname) {
 +        return pathname.getName().endsWith("site.xml");
 +      }
 +    }).length > 0;
 +  }
 +
 +  private void append(StringBuilder classpathBuilder, URL url) throws URISyntaxException {
 +    File file = new File(url.toURI());
 +    // do not include dirs containing hadoop or accumulo site files
 +    if (!containsSiteFile(file))
 +      classpathBuilder.append(File.pathSeparator).append(file.getAbsolutePath());
 +  }
 +
 +  private String getClasspath() throws IOException {
 +
 +    try {
 +      ArrayList<ClassLoader> classloaders = new ArrayList<ClassLoader>();
 +
 +      ClassLoader cl = this.getClass().getClassLoader();
 +
 +      while (cl != null) {
 +        classloaders.add(cl);
 +        cl = cl.getParent();
 +      }
 +
 +      Collections.reverse(classloaders);
 +
 +      StringBuilder classpathBuilder = new StringBuilder();
 +      classpathBuilder.append(config.getConfDir().getAbsolutePath());
 +
 +      if (config.getClasspathItems() == null) {
 +
 +        // assume 0 is the system classloader and skip it
 +        for (int i = 1; i < classloaders.size(); i++) {
 +          ClassLoader classLoader = classloaders.get(i);
 +
 +          if (classLoader instanceof URLClassLoader) {
 +
 +            URLClassLoader ucl = (URLClassLoader) classLoader;
 +
 +            for (URL u : ucl.getURLs()) {
 +              append(classpathBuilder, u);
 +            }
 +
 +          } else if (classLoader instanceof VFSClassLoader) {
 +
 +            VFSClassLoader vcl = (VFSClassLoader) classLoader;
 +            for (FileObject f : vcl.getFileObjects()) {
 +              append(classpathBuilder, f.getURL());
 +            }
 +          } else {
 +            throw new IllegalArgumentException("Unknown classloader type : " + classLoader.getClass().getName());
 +          }
 +        }
 +      } else {
 +        for (String s : config.getClasspathItems())
 +          classpathBuilder.append(File.pathSeparator).append(s);
 +      }
 +
 +      return classpathBuilder.toString();
 +
 +    } catch (URISyntaxException e) {
 +      throw new IOException(e);
 +    }
 +  }
 +
 +  private Process _exec(Class<?> clazz, List<String> extraJvmOpts, String... args) throws IOException {
 +    String javaHome = System.getProperty("java.home");
 +    String javaBin = javaHome + File.separator + "bin" + File.separator + "java";
 +    String classpath = getClasspath();
 +
 +    String className = clazz.getName();
 +
 +    ArrayList<String> argList = new ArrayList<String>();
 +    argList.addAll(Arrays.asList(javaBin, "-Dproc=" + clazz.getSimpleName(), "-cp", classpath));
 +    argList.addAll(extraJvmOpts);
 +    for (Entry<String,String> sysProp : config.getSystemProperties().entrySet()) {
 +      argList.add(String.format("-D%s=%s", sysProp.getKey(), sysProp.getValue()));
 +    }
 +    argList.addAll(Arrays.asList("-XX:+UseConcMarkSweepGC", "-XX:CMSInitiatingOccupancyFraction=75", "-Dapple.awt.UIElement=true", Main.class.getName(), className));
 +    argList.addAll(Arrays.asList(args));
 +
 +    ProcessBuilder builder = new ProcessBuilder(argList);
 +
 +    builder.environment().put("ACCUMULO_HOME", config.getDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_LOG_DIR", config.getLogDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_CLIENT_CONF_PATH", config.getClientConfFile().getAbsolutePath());
 +    String ldLibraryPath = StringUtil.join(Arrays.asList(config.getNativeLibPaths()), File.pathSeparator);
 +    builder.environment().put("LD_LIBRARY_PATH", ldLibraryPath);
 +    builder.environment().put("DYLD_LIBRARY_PATH", ldLibraryPath);
 +
 +    // if we're running under accumulo.start, we forward these env vars
 +    String env = System.getenv("HADOOP_PREFIX");
 +    if (env != null)
 +      builder.environment().put("HADOOP_PREFIX", env);
 +    env = System.getenv("ZOOKEEPER_HOME");
 +    if (env != null)
 +      builder.environment().put("ZOOKEEPER_HOME", env);
 +    builder.environment().put("ACCUMULO_CONF_DIR", config.getConfDir().getAbsolutePath());
 +    // hadoop-2.2 puts error messages in the logs if this is not set
 +    builder.environment().put("HADOOP_HOME", config.getDir().getAbsolutePath());
 +
 +    Process process = builder.start();
 +
 +    LogWriter lw;
 +    lw = new LogWriter(process.getErrorStream(), new File(config.getLogDir(), clazz.getSimpleName() + "_" + process.hashCode() + ".err"));
 +    logWriters.add(lw);
 +    lw.start();
 +    lw = new LogWriter(process.getInputStream(), new File(config.getLogDir(), clazz.getSimpleName() + "_" + process.hashCode() + ".out"));
 +    logWriters.add(lw);
 +    lw.start();
 +
 +    return process;
 +  }
 +
 +  private Process _exec(Class<?> clazz, ServerType serverType, String... args) throws IOException {
 +
 +    List<String> jvmOpts = new ArrayList<String>();
 +    jvmOpts.add("-Xmx" + config.getMemory(serverType));
 +
 +    if (config.isJDWPEnabled()) {
 +      Integer port = PortUtils.getRandomFreePort();
 +      jvmOpts.addAll(buildRemoteDebugParams(port));
 +      debugPorts.add(new Pair<ServerType,Integer>(serverType, port));
 +    }
 +    return _exec(clazz, jvmOpts, args);
 +  }
 +
 +  /**
 +   * 
 +   * @param dir
 +   *          An empty or nonexistant temp directoy that Accumulo and Zookeeper can store data in. Creating the directory is left to the user. Java 7, Guava,
 +   *          and Junit provide methods for creating temporary directories.
 +   * @param rootPassword
 +   *          Initial root password for instance.
 +   */
 +  public MiniAccumuloClusterImpl(File dir, String rootPassword) throws IOException {
 +    this(new MiniAccumuloConfigImpl(dir, rootPassword));
 +  }
 +
 +  /**
 +   * @param config
 +   *          initial configuration
 +   */
 +  @SuppressWarnings("deprecation")
 +  public MiniAccumuloClusterImpl(MiniAccumuloConfigImpl config) throws IOException {
 +
 +    this.config = config.initialize();
 +
 +    config.getConfDir().mkdirs();
 +    config.getAccumuloDir().mkdirs();
 +    config.getZooKeeperDir().mkdirs();
 +    config.getLogDir().mkdirs();
 +    config.getWalogDir().mkdirs();
 +    config.getLibDir().mkdirs();
 +    config.getLibExtDir().mkdirs();
 +
 +    if (config.useMiniDFS()) {
 +      File nn = new File(config.getAccumuloDir(), "nn");
 +      nn.mkdirs();
 +      File dn = new File(config.getAccumuloDir(), "dn");
 +      dn.mkdirs();
 +      File dfs = new File(config.getAccumuloDir(), "dfs");
 +      dfs.mkdirs();
 +      Configuration conf = new Configuration();
 +      conf.set(DFSConfigKeys.DFS_NAMENODE_NAME_DIR_KEY, nn.getAbsolutePath());
 +      conf.set(DFSConfigKeys.DFS_DATANODE_DATA_DIR_KEY, dn.getAbsolutePath());
 +      conf.set(DFSConfigKeys.DFS_REPLICATION_KEY, "1");
 +      conf.set("dfs.support.append", "true");
 +      conf.set("dfs.datanode.synconclose", "true");
 +      conf.set("dfs.datanode.data.dir.perm", MiniDFSUtil.computeDatanodeDirectoryPermission());
 +      String oldTestBuildData = System.setProperty("test.build.data", dfs.getAbsolutePath());
 +      miniDFS = new MiniDFSCluster(conf, 1, true, null);
 +      if (oldTestBuildData == null)
 +        System.clearProperty("test.build.data");
 +      else
 +        System.setProperty("test.build.data", oldTestBuildData);
 +      miniDFS.waitClusterUp();
 +      InetSocketAddress dfsAddress = miniDFS.getNameNode().getNameNodeAddress();
 +      dfsUri = "hdfs://" + dfsAddress.getHostName() + ":" + dfsAddress.getPort();
 +      File coreFile = new File(config.getConfDir(), "core-site.xml");
 +      writeConfig(coreFile, Collections.singletonMap("fs.default.name", dfsUri).entrySet());
 +      File hdfsFile = new File(config.getConfDir(), "hdfs-site.xml");
 +      writeConfig(hdfsFile, conf);
 +
 +      Map<String,String> siteConfig = config.getSiteConfig();
 +      siteConfig.put(Property.INSTANCE_DFS_URI.getKey(), dfsUri);
 +      siteConfig.put(Property.INSTANCE_DFS_DIR.getKey(), "/accumulo");
 +      config.setSiteConfig(siteConfig);
 +    } else {
 +      dfsUri = "file://";
 +    }
 +
 +    File clientConfFile = config.getClientConfFile();
 +    // Write only the properties that correspond to ClientConfiguration properties
 +    writeConfigProperties(clientConfFile, Maps.filterEntries(config.getSiteConfig(), new Predicate<Entry<String,String>>() {
 +      @Override
 +      public boolean apply(Entry<String,String> v) {
 +        return ClientConfiguration.ClientProperty.getPropertyByKey(v.getKey()) != null;
 +      }
 +    }));
 +
 +    File siteFile = new File(config.getConfDir(), "accumulo-site.xml");
 +    writeConfig(siteFile, config.getSiteConfig().entrySet());
 +
 +    zooCfgFile = new File(config.getConfDir(), "zoo.cfg");
 +    FileWriter fileWriter = new FileWriter(zooCfgFile);
 +
 +    // zookeeper uses Properties to read its config, so use that to write in order to properly escape things like Windows paths
 +    Properties zooCfg = new Properties();
 +    zooCfg.setProperty("tickTime", "2000");
 +    zooCfg.setProperty("initLimit", "10");
 +    zooCfg.setProperty("syncLimit", "5");
 +    zooCfg.setProperty("clientPort", config.getZooKeeperPort() + "");
 +    zooCfg.setProperty("maxClientCnxns", "1000");
 +    zooCfg.setProperty("dataDir", config.getZooKeeperDir().getAbsolutePath());
 +    zooCfg.store(fileWriter, null);
 +
 +    fileWriter.close();
 +  }
 +
 +  private void writeConfig(File file, Iterable<Map.Entry<String,String>> settings) throws IOException {
 +    FileWriter fileWriter = new FileWriter(file);
 +    fileWriter.append("<configuration>\n");
 +
 +    for (Entry<String,String> entry : settings) {
 +      String value = entry.getValue().replace("&", "&amp;").replace("<", "&lt;").replace(">", "&gt;");
 +      fileWriter.append("<property><name>" + entry.getKey() + "</name><value>" + value + "</value></property>\n");
 +    }
 +    fileWriter.append("</configuration>\n");
 +    fileWriter.close();
 +  }
 +
 +  private void writeConfigProperties(File file, Map<String,String> settings) throws IOException {
 +    FileWriter fileWriter = new FileWriter(file);
 +
 +    for (Entry<String,String> entry : settings.entrySet())
 +      fileWriter.append(entry.getKey() + "=" + entry.getValue() + "\n");
 +    fileWriter.close();
 +  }
 +
 +  /**
 +   * Starts Accumulo and Zookeeper processes. Can only be called once.
 +   * 
 +   * @throws IllegalStateException
 +   *           if already started
 +   */
 +  public void start() throws IOException, InterruptedException {
 +
 +    if (!initialized) {
 +
 +      Runtime.getRuntime().addShutdownHook(new Thread() {
 +        @Override
 +        public void run() {
 +          try {
 +            MiniAccumuloClusterImpl.this.stop();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          } catch (InterruptedException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      });
 +    }
 +
 +    if (zooKeeperProcess == null) {
 +      zooKeeperProcess = _exec(ZooKeeperServerMain.class, ServerType.ZOOKEEPER, zooCfgFile.getAbsolutePath());
 +    }
 +
 +    if (!initialized) {
 +      // sleep a little bit to let zookeeper come up before calling init, seems to work better
 +      long startTime = System.currentTimeMillis();
 +      while (true) {
 +        Socket s = null;
 +        try {
 +          s = new Socket("localhost", config.getZooKeeperPort());
 +          s.getOutputStream().write("ruok\n".getBytes());
 +          s.getOutputStream().flush();
 +          byte buffer[] = new byte[100];
 +          int n = s.getInputStream().read(buffer);
 +          if (n >= 4 && new String(buffer, 0, 4).equals("imok"))
 +            break;
 +        } catch (Exception e) {
 +          if (System.currentTimeMillis() - startTime >= ZOOKEEPER_STARTUP_WAIT) {
 +            throw new RuntimeException("Zookeeper did not start within " + (ZOOKEEPER_STARTUP_WAIT/1000) + " seconds. Check the logs in " + config.getLogDir() + " for errors.  Last exception: " + e);
 +          }
 +          UtilWaitThread.sleep(250);
 +        } finally {
 +          if (s != null)
 +            s.close();
 +        }
 +      }
 +      Process initProcess = exec(Initialize.class, "--instance-name", config.getInstanceName(), "--password", config.getRootPassword());
 +      int ret = initProcess.waitFor();
 +      if (ret != 0) {
 +        throw new RuntimeException("Initialize process returned " + ret + ". Check the logs in " + config.getLogDir() + " for errors.");
 +      }
 +      initialized = true;
 +    }
 +    synchronized (tabletServerProcesses) {
 +      for (int i = tabletServerProcesses.size(); i < config.getNumTservers(); i++) {
 +        tabletServerProcesses.add(_exec(TabletServer.class, ServerType.TABLET_SERVER));
 +      }
 +    }
 +    int ret = 0;
 +    for (int i = 0; i < 5; i++) {
 +      ret = exec(Main.class, SetGoalState.class.getName(), MasterGoalState.NORMAL.toString()).waitFor();
 +      if (ret == 0)
 +        break;
 +      UtilWaitThread.sleep(1000);
 +    }
 +    if (ret != 0) {
 +      throw new RuntimeException("Could not set master goal state, process returned " + ret + ". Check the logs in " + config.getLogDir() + " for errors.");
 +    }
 +    if (masterProcess == null) {
 +      masterProcess = _exec(Master.class, ServerType.MASTER);
 +    }
 +
 +    if (gcProcess == null) {
 +      gcProcess = _exec(SimpleGarbageCollector.class, ServerType.GARBAGE_COLLECTOR);
 +    }
 +  }
 +
 +  private List<String> buildRemoteDebugParams(int port) {
 +    return Arrays.asList(new String[] {"-Xdebug", String.format("-Xrunjdwp:transport=dt_socket,server=y,suspend=n,address=%d", port)});
 +  }
 +
 +  /**
 +   * @return generated remote debug ports if in debug mode.
 +   * @since 1.6.0
 +   */
 +  public Set<Pair<ServerType,Integer>> getDebugPorts() {
 +    return debugPorts;
 +  }
 +
 +  List<ProcessReference> references(Process... procs) {
 +    List<ProcessReference> result = new ArrayList<ProcessReference>();
 +    for (Process proc : procs) {
 +      result.add(new ProcessReference(proc));
 +    }
 +    return result;
 +  }
 +
 +  public Map<ServerType,Collection<ProcessReference>> getProcesses() {
 +    Map<ServerType,Collection<ProcessReference>> result = new HashMap<ServerType,Collection<ProcessReference>>();
 +    result.put(ServerType.MASTER, references(masterProcess));
 +    result.put(ServerType.TABLET_SERVER, references(tabletServerProcesses.toArray(new Process[0])));
 +    result.put(ServerType.ZOOKEEPER, references(zooKeeperProcess));
 +    if (null != gcProcess) {
 +      result.put(ServerType.GARBAGE_COLLECTOR, references(gcProcess));
 +    }
 +    return result;
 +  }
 +
 +  public void killProcess(ServerType type, ProcessReference proc) throws ProcessNotFoundException, InterruptedException {
 +    boolean found = false;
 +    switch (type) {
 +      case MASTER:
 +        if (proc.equals(masterProcess)) {
 +          masterProcess.destroy();
++          masterProcess.waitFor();
 +          masterProcess = null;
 +          found = true;
 +        }
 +        break;
 +      case TABLET_SERVER:
 +        synchronized (tabletServerProcesses) {
 +          for (Process tserver : tabletServerProcesses) {
 +            if (proc.equals(tserver)) {
 +              tabletServerProcesses.remove(tserver);
 +              tserver.destroy();
++              tserver.waitFor();
 +              found = true;
 +              break;
 +            }
 +          }
 +        }
 +        break;
 +      case ZOOKEEPER:
 +        if (proc.equals(zooKeeperProcess)) {
 +          zooKeeperProcess.destroy();
++          zooKeeperProcess.waitFor();
 +          zooKeeperProcess = null;
 +          found = true;
 +        }
 +        break;
 +      case GARBAGE_COLLECTOR:
 +        if (proc.equals(gcProcess)) {
 +          gcProcess.destroy();
++          gcProcess.waitFor();
 +          gcProcess = null;
 +          found = true;
 +        }
 +        break;
 +    }
 +    if (!found)
 +      throw new ProcessNotFoundException();
 +  }
 +
 +  /**
 +   * @return Accumulo instance name
 +   */
 +  public String getInstanceName() {
 +    return config.getInstanceName();
 +  }
 +
 +  /**
 +   * @return zookeeper connection string
 +   */
 +  public String getZooKeepers() {
 +    return config.getZooKeepers();
 +  }
 +
 +  /**
 +   * Stops Accumulo and Zookeeper processes. If stop is not called, there is a shutdown hook that is setup to kill the processes. However its probably best to
 +   * call stop in a finally block as soon as possible.
 +   */
 +  public void stop() throws IOException, InterruptedException {
 +    for (LogWriter lw : logWriters) {
 +      lw.flush();
 +    }
 +
 +    if (zooKeeperProcess != null) {
 +      zooKeeperProcess.destroy();
++      zooKeeperProcess.waitFor();
 +    }
 +    if (masterProcess != null) {
 +      masterProcess.destroy();
++      masterProcess.waitFor();
 +    }
 +    if (tabletServerProcesses != null) {
 +      synchronized (tabletServerProcesses) {
 +        for (Process tserver : tabletServerProcesses) {
 +          tserver.destroy();
++          tserver.waitFor();
 +        }
 +      }
 +    }
 +    if (gcProcess != null) {
 +      gcProcess.destroy();
++      gcProcess.waitFor();
 +    }
 +
 +    zooKeeperProcess = null;
 +    masterProcess = null;
 +    gcProcess = null;
 +    tabletServerProcesses.clear();
 +    if (config.useMiniDFS() && miniDFS != null)
 +      miniDFS.shutdown();
-     for (Process p : cleanup)
++    for (Process p : cleanup) {
 +      p.destroy();
++      p.waitFor();
++    }
 +    miniDFS = null;
 +  }
 +
 +  /**
 +   * @since 1.6.0
 +   */
 +  public MiniAccumuloConfigImpl getConfig() {
 +    return config;
 +  }
 +
 +  /**
 +   * Utility method to get a connector to the MAC.
 +   * 
 +   * @since 1.6.0
 +   */
 +  public Connector getConnector(String user, String passwd) throws AccumuloException, AccumuloSecurityException {
 +    Instance instance = new ZooKeeperInstance(getClientConfig());
 +    return instance.getConnector(user, new PasswordToken(passwd));
 +  }
 +
 +  public ClientConfiguration getClientConfig() {
 +    return new ClientConfiguration(Arrays.asList(new MapConfiguration(config.getSiteConfig()))).withInstance(this.getInstanceName()).withZkHosts(
 +        this.getZooKeepers());
 +  }
 +
 +  public FileSystem getFileSystem() {
 +    try {
 +      return FileSystem.get(new URI(dfsUri), new Configuration());
 +    } catch (Exception e) {
 +      throw new RuntimeException(e);
 +    }
 +  }
 +}


[06/10] git commit: Merge branch '1.4.5-SNAPSHOT' into 1.5.2-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.4.5-SNAPSHOT' into 1.5.2-SNAPSHOT

Conflicts:
	minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/63d5e55a
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/63d5e55a
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/63d5e55a

Branch: refs/heads/1.5.2-SNAPSHOT
Commit: 63d5e55a0b03910246b9b21efecfde5ac5e709f0
Parents: 4b7a011 0753a75
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:50:59 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:50:59 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java      | 13 ++++++++++---
 1 file changed, 10 insertions(+), 3 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/63d5e55a/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --cc minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index 4018a33,0000000..a366c16
mode 100644,000000..100644
--- a/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@@ -1,385 -1,0 +1,392 @@@
 +/*
 + * Licensed to the Apache Software Foundation (ASF) under one or more
 + * contributor license agreements.  See the NOTICE file distributed with
 + * this work for additional information regarding copyright ownership.
 + * The ASF licenses this file to You under the Apache License, Version 2.0
 + * (the "License"); you may not use this file except in compliance with
 + * the License.  You may obtain a copy of the License at
 + *
 + *     http://www.apache.org/licenses/LICENSE-2.0
 + *
 + * Unless required by applicable law or agreed to in writing, software
 + * distributed under the License is distributed on an "AS IS" BASIS,
 + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
 + * See the License for the specific language governing permissions and
 + * limitations under the License.
 + */
 +package org.apache.accumulo.minicluster;
 +
 +import java.io.BufferedReader;
 +import java.io.BufferedWriter;
 +import java.io.File;
 +import java.io.FileOutputStream;
 +import java.io.IOException;
 +import java.io.InputStream;
 +import java.io.InputStreamReader;
 +import java.io.OutputStreamWriter;
 +import java.io.Writer;
 +import java.util.ArrayList;
 +import java.util.Arrays;
 +import java.util.HashMap;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Properties;
 +
 +import org.apache.accumulo.core.Constants;
 +import org.apache.accumulo.core.conf.Property;
 +import org.apache.accumulo.core.util.UtilWaitThread;
 +import org.apache.accumulo.server.gc.SimpleGarbageCollector;
 +import org.apache.accumulo.server.master.Master;
 +import org.apache.accumulo.server.tabletserver.TabletServer;
 +import org.apache.accumulo.server.util.Initialize;
 +import org.apache.accumulo.server.util.PortUtils;
 +import org.apache.accumulo.server.util.time.SimpleTimer;
 +import org.apache.accumulo.start.Main;
 +import org.apache.zookeeper.server.ZooKeeperServerMain;
 +
 +/**
 + * A utility class that will create Zookeeper and Accumulo processes that write all of their data to a single local directory. This class makes it easy to test
 + * code against a real Accumulo instance. Its much more accurate for testing than MockAccumulo, but much slower than MockAccumulo.
 + * 
 + * @since 1.5.0
 + */
 +public class MiniAccumuloCluster {
 +  
 +  private static final String INSTANCE_SECRET = "DONTTELL";
 +  private static final String INSTANCE_NAME = "miniInstance";
 +  
 +  private static class LogWriter extends Thread {
 +    private BufferedReader in;
 +    private BufferedWriter out;
 +    
 +    /**
 +     * @throws IOException
 +     */
 +    public LogWriter(InputStream stream, File logFile) throws IOException {
 +      this.setDaemon(true);
 +      this.in = new BufferedReader(new InputStreamReader(stream, Constants.UTF8));
 +      out = new BufferedWriter(new OutputStreamWriter(new FileOutputStream(logFile), Constants.UTF8));
 +      
 +      SimpleTimer.getInstance().schedule(new Runnable() {
 +        @Override
 +        public void run() {
 +          try {
 +            flush();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      }, 1000, 1000);
 +    }
 +    
 +    public synchronized void flush() throws IOException {
 +      if (out != null)
 +        out.flush();
 +    }
 +    
 +    @Override
 +    public void run() {
 +      String line;
 +      
 +      try {
 +        while ((line = in.readLine()) != null) {
 +          out.append(line);
 +          out.append("\n");
 +        }
 +        
 +        synchronized (this) {
 +          out.close();
 +          out = null;
 +          in.close();
 +        }
 +        
 +      } catch (IOException e) {}
 +    }
 +  }
 +  
 +  private File libDir;
 +  private File libExtDir;
 +  private File confDir;
 +  private File zooKeeperDir;
 +  private File accumuloDir;
 +  private File zooCfgFile;
 +  private File logDir;
 +  private File walogDir;
 +  
 +  private Process zooKeeperProcess;
 +  private Process masterProcess;
 +  private Process gcProcess;
 +  
 +  private int zooKeeperPort;
 +  
 +  private List<LogWriter> logWriters = new ArrayList<MiniAccumuloCluster.LogWriter>();
 +  
 +  private MiniAccumuloConfig config;
 +  private Process[] tabletServerProcesses;
 +  
 +  private Process exec(Class<? extends Object> clazz, String... args) throws IOException {
 +    String javaHome = System.getProperty("java.home");
 +    String javaBin = javaHome + File.separator + "bin" + File.separator + "java";
 +    String classpath = System.getProperty("java.class.path");
 +    
 +    classpath = confDir.getAbsolutePath() + File.pathSeparator + classpath;
 +    
 +    String className = clazz.getCanonicalName();
 +    
 +    ArrayList<String> argList = new ArrayList<String>();
 +    
 +    argList.addAll(Arrays.asList(javaBin, "-cp", classpath, "-Xmx128m", "-XX:+UseConcMarkSweepGC", "-XX:CMSInitiatingOccupancyFraction=75",
 +        "-Dapple.awt.UIElement=true", Main.class.getName(), className));
 +    
 +    argList.addAll(Arrays.asList(args));
 +    
 +    ProcessBuilder builder = new ProcessBuilder(argList);
 +    
 +    builder.environment().put("ACCUMULO_HOME", config.getDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_LOG_DIR", logDir.getAbsolutePath());
 +    
 +    // if we're running under accumulo.start, we forward these env vars
 +    String env = System.getenv("HADOOP_PREFIX");
 +    if (env != null)
 +      builder.environment().put("HADOOP_PREFIX", env);
 +    env = System.getenv("ZOOKEEPER_HOME");
 +    if (env != null)
 +      builder.environment().put("ZOOKEEPER_HOME", env);
 +    
 +    Process process = builder.start();
 +    
 +    LogWriter lw;
 +    lw = new LogWriter(process.getErrorStream(), new File(logDir, clazz.getSimpleName() + "_" + process.hashCode() + ".err"));
 +    logWriters.add(lw);
 +    lw.start();
 +    lw = new LogWriter(process.getInputStream(), new File(logDir, clazz.getSimpleName() + "_" + process.hashCode() + ".out"));
 +    logWriters.add(lw);
 +    lw.start();
 +    
 +    return process;
 +  }
 +  
 +  private void appendProp(Writer fileWriter, Property key, String value, Map<String,String> siteConfig) throws IOException {
 +    appendProp(fileWriter, key.getKey(), value, siteConfig);
 +  }
 +  
 +  private void appendProp(Writer fileWriter, String key, String value, Map<String,String> siteConfig) throws IOException {
 +    if (!siteConfig.containsKey(key))
 +      fileWriter.append("<property><name>" + key + "</name><value>" + value + "</value></property>\n");
 +  }
 +
 +  /**
 +   * Sets a given key with a random port for the value on the site config if it doesn't already exist.
 +   */
 +  private void mergePropWithRandomPort(Map<String,String> siteConfig, String key) {
 +    if (!siteConfig.containsKey(key)) {
 +      siteConfig.put(key, "0");
 +    }
 +  }
 +  
 +  /**
 +   * 
 +   * @param dir
 +   *          An empty or nonexistant temp directoy that Accumulo and Zookeeper can store data in. Creating the directory is left to the user. Java 7, Guava,
 +   *          and Junit provide methods for creating temporary directories.
 +   * @param rootPassword
 +   *          Initial root password for instance.
 +   * @throws IOException
 +   */
 +  public MiniAccumuloCluster(File dir, String rootPassword) throws IOException {
 +    this(new MiniAccumuloConfig(dir, rootPassword));
 +  }
 +  
 +  /**
 +   * @param config
 +   *          initial configuration
 +   * @throws IOException
 +   */
 +  
 +  public MiniAccumuloCluster(MiniAccumuloConfig config) throws IOException {
 +    
 +    if (config.getDir().exists() && !config.getDir().isDirectory())
 +      throw new IllegalArgumentException("Must pass in directory, " + config.getDir() + " is a file");
 +    
 +    if (config.getDir().exists() && config.getDir().list().length != 0)
 +      throw new IllegalArgumentException("Directory " + config.getDir() + " is not empty");
 +    
 +    this.config = config;
 +    
 +    libDir = new File(config.getDir(), "lib");
 +    libExtDir = new File(libDir, "ext");
 +    confDir = new File(config.getDir(), "conf");
 +    accumuloDir = new File(config.getDir(), "accumulo");
 +    zooKeeperDir = new File(config.getDir(), "zookeeper");
 +    logDir = new File(config.getDir(), "logs");
 +    walogDir = new File(config.getDir(), "walogs");
 +    
 +    confDir.mkdirs();
 +    accumuloDir.mkdirs();
 +    zooKeeperDir.mkdirs();
 +    logDir.mkdirs();
 +    walogDir.mkdirs();
 +    libDir.mkdirs();
 +    
 +    // Avoid the classloader yelling that the general.dynamic.classpaths value is invalid because
 +    // $ACCUMULO_HOME/lib/ext isn't defined.
 +    libExtDir.mkdirs();
 +    
 +    zooKeeperPort = PortUtils.getRandomFreePort();
 +    
 +    File siteFile = new File(confDir, "accumulo-site.xml");
 +    
 +    OutputStreamWriter fileWriter = new OutputStreamWriter(new FileOutputStream(siteFile), Constants.UTF8);
 +    fileWriter.append("<configuration>\n");
 +    
 +    HashMap<String,String> siteConfig = new HashMap<String,String>(config.getSiteConfig());
 +    
 +    appendProp(fileWriter, Property.INSTANCE_DFS_URI, "file:///", siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_DFS_DIR, accumuloDir.getAbsolutePath(), siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_ZK_HOST, "localhost:" + zooKeeperPort, siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_SECRET, INSTANCE_SECRET, siteConfig);
 +    appendProp(fileWriter, Property.TSERV_PORTSEARCH, "true", siteConfig);
 +    appendProp(fileWriter, Property.LOGGER_DIR, walogDir.getAbsolutePath(), siteConfig);
 +    appendProp(fileWriter, Property.TSERV_DATACACHE_SIZE, "10M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_INDEXCACHE_SIZE, "10M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_MAXMEM, "50M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_WALOG_MAX_SIZE, "100M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_NATIVEMAP_ENABLED, "false", siteConfig);
 +    appendProp(fileWriter, Property.TRACE_TOKEN_PROPERTY_PREFIX + ".password", config.getRootPassword(), siteConfig);
 +    appendProp(fileWriter, Property.GC_CYCLE_DELAY, "4s", siteConfig);
 +    appendProp(fileWriter, Property.GC_CYCLE_START, "0s", siteConfig);
 +    mergePropWithRandomPort(siteConfig, Property.MASTER_CLIENTPORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.TRACE_PORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.TSERV_CLIENTPORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.MONITOR_PORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.GC_PORT.getKey());
 +    
 +    // since there is a small amount of memory, check more frequently for majc... setting may not be needed in 1.5
 +    appendProp(fileWriter, Property.TSERV_MAJC_DELAY, "3", siteConfig);
 +    
 +    // ACCUMULO-1472 -- Use the classpath, not what might be installed on the system.
 +    // We have to set *something* here, otherwise the AccumuloClassLoader will default to pulling from 
 +    // environment variables (e.g. ACCUMULO_HOME, HADOOP_HOME/PREFIX) which will result in multiple copies
 +    // of artifacts on the classpath as they'll be provided by the invoking application
 +    appendProp(fileWriter, Property.GENERAL_CLASSPATHS, libDir.getAbsolutePath() + "/[^.].*.jar", siteConfig);
 +    appendProp(fileWriter, Property.GENERAL_DYNAMIC_CLASSPATHS, libExtDir.getAbsolutePath() + "/[^.].*.jar", siteConfig);
 +
 +    for (Entry<String,String> entry : siteConfig.entrySet())
 +      fileWriter.append("<property><name>" + entry.getKey() + "</name><value>" + entry.getValue() + "</value></property>\n");
 +    fileWriter.append("</configuration>\n");
 +    fileWriter.close();
 +    
 +    zooCfgFile = new File(confDir, "zoo.cfg");
 +    fileWriter = new OutputStreamWriter(new FileOutputStream(zooCfgFile), Constants.UTF8);
 +    
 +    // zookeeper uses Properties to read its config, so use that to write in order to properly escape things like Windows paths
 +    Properties zooCfg = new Properties();
 +    zooCfg.setProperty("tickTime", "1000");
 +    zooCfg.setProperty("initLimit", "10");
 +    zooCfg.setProperty("syncLimit", "5");
 +    zooCfg.setProperty("clientPort", zooKeeperPort + "");
 +    zooCfg.setProperty("maxClientCnxns", "100");
 +    zooCfg.setProperty("dataDir", zooKeeperDir.getAbsolutePath());
 +    zooCfg.store(fileWriter, null);
 +    
 +    fileWriter.close();
 +    
 +  }
 +  
 +  /**
 +   * Starts Accumulo and Zookeeper processes. Can only be called once.
 +   * 
 +   * @throws IOException
 +   * @throws InterruptedException
 +   * @throws IllegalStateException
 +   *           if already started
 +   */
 +  
 +  public void start() throws IOException, InterruptedException {
 +    if (zooKeeperProcess != null)
 +      throw new IllegalStateException("Already started");
 +    
 +    Runtime.getRuntime().addShutdownHook(new Thread() {
 +      @Override
 +      public void run() {
 +        try {
 +          MiniAccumuloCluster.this.stop();
 +        } catch (IOException e) {
 +          e.printStackTrace();
 +        } catch (InterruptedException e) {
 +          e.printStackTrace();
 +        }
 +      }
 +    });
 +    
 +    zooKeeperProcess = exec(Main.class, ZooKeeperServerMain.class.getName(), zooCfgFile.getAbsolutePath());
 +    
 +    // sleep a little bit to let zookeeper come up before calling init, seems to work better
 +    UtilWaitThread.sleep(250);
 +    
 +    Process initProcess = exec(Initialize.class, "--instance-name", INSTANCE_NAME, "--password", config.getRootPassword());
 +    int ret = initProcess.waitFor();
 +    if (ret != 0) {
 +      throw new RuntimeException("Initialize process returned " + ret + ". Check the logs in " + logDir + " for errors.");
 +    }
 +    
 +    tabletServerProcesses = new Process[config.getNumTservers()];
 +    for (int i = 0; i < config.getNumTservers(); i++) {
 +      tabletServerProcesses[i] = exec(TabletServer.class);
 +    }
 +    
 +    masterProcess = exec(Master.class);
 +    
 +    gcProcess = exec(SimpleGarbageCollector.class);
 +  }
 +  
 +  /**
 +   * @return Accumulo instance name
 +   */
 +  
 +  public String getInstanceName() {
 +    return INSTANCE_NAME;
 +  }
 +  
 +  /**
 +   * @return zookeeper connection string
 +   */
 +  
 +  public String getZooKeepers() {
 +    return "localhost:" + zooKeeperPort;
 +  }
 +  
 +  /**
 +   * Stops Accumulo and Zookeeper processes. If stop is not called, there is a shutdown hook that is setup to kill the processes. Howerver its probably best to
 +   * call stop in a finally block as soon as possible.
 +   * 
 +   * @throws IOException
 +   * @throws InterruptedException
 +   */
 +  
 +  public void stop() throws IOException, InterruptedException {
-     if (zooKeeperProcess != null)
++    if (zooKeeperProcess != null) {
 +      zooKeeperProcess.destroy();
-     if (masterProcess != null)
++      zooKeeperProcess.waitFor();
++    }
++    if (masterProcess != null) {
 +      masterProcess.destroy();
++      masterProcess.waitFor();
++    }
 +    if (tabletServerProcesses != null) {
 +      for (Process tserver : tabletServerProcesses) {
 +        tserver.destroy();
++        tserver.waitFor();
 +      }
 +    }
 +    
 +    for (LogWriter lw : logWriters)
 +      lw.flush();
 +
-     if (gcProcess != null)
++    if (gcProcess != null) {
 +      gcProcess.destroy();
++      gcProcess.waitFor();
++    }
 +  }
 +}


[05/10] git commit: Merge branch '1.4.5-SNAPSHOT' into 1.5.2-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.4.5-SNAPSHOT' into 1.5.2-SNAPSHOT

Conflicts:
	minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/63d5e55a
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/63d5e55a
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/63d5e55a

Branch: refs/heads/1.6.0-SNAPSHOT
Commit: 63d5e55a0b03910246b9b21efecfde5ac5e709f0
Parents: 4b7a011 0753a75
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:50:59 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:50:59 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java      | 13 ++++++++++---
 1 file changed, 10 insertions(+), 3 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/63d5e55a/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --cc minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index 4018a33,0000000..a366c16
mode 100644,000000..100644
--- a/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@@ -1,385 -1,0 +1,392 @@@
 +/*
 + * Licensed to the Apache Software Foundation (ASF) under one or more
 + * contributor license agreements.  See the NOTICE file distributed with
 + * this work for additional information regarding copyright ownership.
 + * The ASF licenses this file to You under the Apache License, Version 2.0
 + * (the "License"); you may not use this file except in compliance with
 + * the License.  You may obtain a copy of the License at
 + *
 + *     http://www.apache.org/licenses/LICENSE-2.0
 + *
 + * Unless required by applicable law or agreed to in writing, software
 + * distributed under the License is distributed on an "AS IS" BASIS,
 + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
 + * See the License for the specific language governing permissions and
 + * limitations under the License.
 + */
 +package org.apache.accumulo.minicluster;
 +
 +import java.io.BufferedReader;
 +import java.io.BufferedWriter;
 +import java.io.File;
 +import java.io.FileOutputStream;
 +import java.io.IOException;
 +import java.io.InputStream;
 +import java.io.InputStreamReader;
 +import java.io.OutputStreamWriter;
 +import java.io.Writer;
 +import java.util.ArrayList;
 +import java.util.Arrays;
 +import java.util.HashMap;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Properties;
 +
 +import org.apache.accumulo.core.Constants;
 +import org.apache.accumulo.core.conf.Property;
 +import org.apache.accumulo.core.util.UtilWaitThread;
 +import org.apache.accumulo.server.gc.SimpleGarbageCollector;
 +import org.apache.accumulo.server.master.Master;
 +import org.apache.accumulo.server.tabletserver.TabletServer;
 +import org.apache.accumulo.server.util.Initialize;
 +import org.apache.accumulo.server.util.PortUtils;
 +import org.apache.accumulo.server.util.time.SimpleTimer;
 +import org.apache.accumulo.start.Main;
 +import org.apache.zookeeper.server.ZooKeeperServerMain;
 +
 +/**
 + * A utility class that will create Zookeeper and Accumulo processes that write all of their data to a single local directory. This class makes it easy to test
 + * code against a real Accumulo instance. Its much more accurate for testing than MockAccumulo, but much slower than MockAccumulo.
 + * 
 + * @since 1.5.0
 + */
 +public class MiniAccumuloCluster {
 +  
 +  private static final String INSTANCE_SECRET = "DONTTELL";
 +  private static final String INSTANCE_NAME = "miniInstance";
 +  
 +  private static class LogWriter extends Thread {
 +    private BufferedReader in;
 +    private BufferedWriter out;
 +    
 +    /**
 +     * @throws IOException
 +     */
 +    public LogWriter(InputStream stream, File logFile) throws IOException {
 +      this.setDaemon(true);
 +      this.in = new BufferedReader(new InputStreamReader(stream, Constants.UTF8));
 +      out = new BufferedWriter(new OutputStreamWriter(new FileOutputStream(logFile), Constants.UTF8));
 +      
 +      SimpleTimer.getInstance().schedule(new Runnable() {
 +        @Override
 +        public void run() {
 +          try {
 +            flush();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      }, 1000, 1000);
 +    }
 +    
 +    public synchronized void flush() throws IOException {
 +      if (out != null)
 +        out.flush();
 +    }
 +    
 +    @Override
 +    public void run() {
 +      String line;
 +      
 +      try {
 +        while ((line = in.readLine()) != null) {
 +          out.append(line);
 +          out.append("\n");
 +        }
 +        
 +        synchronized (this) {
 +          out.close();
 +          out = null;
 +          in.close();
 +        }
 +        
 +      } catch (IOException e) {}
 +    }
 +  }
 +  
 +  private File libDir;
 +  private File libExtDir;
 +  private File confDir;
 +  private File zooKeeperDir;
 +  private File accumuloDir;
 +  private File zooCfgFile;
 +  private File logDir;
 +  private File walogDir;
 +  
 +  private Process zooKeeperProcess;
 +  private Process masterProcess;
 +  private Process gcProcess;
 +  
 +  private int zooKeeperPort;
 +  
 +  private List<LogWriter> logWriters = new ArrayList<MiniAccumuloCluster.LogWriter>();
 +  
 +  private MiniAccumuloConfig config;
 +  private Process[] tabletServerProcesses;
 +  
 +  private Process exec(Class<? extends Object> clazz, String... args) throws IOException {
 +    String javaHome = System.getProperty("java.home");
 +    String javaBin = javaHome + File.separator + "bin" + File.separator + "java";
 +    String classpath = System.getProperty("java.class.path");
 +    
 +    classpath = confDir.getAbsolutePath() + File.pathSeparator + classpath;
 +    
 +    String className = clazz.getCanonicalName();
 +    
 +    ArrayList<String> argList = new ArrayList<String>();
 +    
 +    argList.addAll(Arrays.asList(javaBin, "-cp", classpath, "-Xmx128m", "-XX:+UseConcMarkSweepGC", "-XX:CMSInitiatingOccupancyFraction=75",
 +        "-Dapple.awt.UIElement=true", Main.class.getName(), className));
 +    
 +    argList.addAll(Arrays.asList(args));
 +    
 +    ProcessBuilder builder = new ProcessBuilder(argList);
 +    
 +    builder.environment().put("ACCUMULO_HOME", config.getDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_LOG_DIR", logDir.getAbsolutePath());
 +    
 +    // if we're running under accumulo.start, we forward these env vars
 +    String env = System.getenv("HADOOP_PREFIX");
 +    if (env != null)
 +      builder.environment().put("HADOOP_PREFIX", env);
 +    env = System.getenv("ZOOKEEPER_HOME");
 +    if (env != null)
 +      builder.environment().put("ZOOKEEPER_HOME", env);
 +    
 +    Process process = builder.start();
 +    
 +    LogWriter lw;
 +    lw = new LogWriter(process.getErrorStream(), new File(logDir, clazz.getSimpleName() + "_" + process.hashCode() + ".err"));
 +    logWriters.add(lw);
 +    lw.start();
 +    lw = new LogWriter(process.getInputStream(), new File(logDir, clazz.getSimpleName() + "_" + process.hashCode() + ".out"));
 +    logWriters.add(lw);
 +    lw.start();
 +    
 +    return process;
 +  }
 +  
 +  private void appendProp(Writer fileWriter, Property key, String value, Map<String,String> siteConfig) throws IOException {
 +    appendProp(fileWriter, key.getKey(), value, siteConfig);
 +  }
 +  
 +  private void appendProp(Writer fileWriter, String key, String value, Map<String,String> siteConfig) throws IOException {
 +    if (!siteConfig.containsKey(key))
 +      fileWriter.append("<property><name>" + key + "</name><value>" + value + "</value></property>\n");
 +  }
 +
 +  /**
 +   * Sets a given key with a random port for the value on the site config if it doesn't already exist.
 +   */
 +  private void mergePropWithRandomPort(Map<String,String> siteConfig, String key) {
 +    if (!siteConfig.containsKey(key)) {
 +      siteConfig.put(key, "0");
 +    }
 +  }
 +  
 +  /**
 +   * 
 +   * @param dir
 +   *          An empty or nonexistant temp directoy that Accumulo and Zookeeper can store data in. Creating the directory is left to the user. Java 7, Guava,
 +   *          and Junit provide methods for creating temporary directories.
 +   * @param rootPassword
 +   *          Initial root password for instance.
 +   * @throws IOException
 +   */
 +  public MiniAccumuloCluster(File dir, String rootPassword) throws IOException {
 +    this(new MiniAccumuloConfig(dir, rootPassword));
 +  }
 +  
 +  /**
 +   * @param config
 +   *          initial configuration
 +   * @throws IOException
 +   */
 +  
 +  public MiniAccumuloCluster(MiniAccumuloConfig config) throws IOException {
 +    
 +    if (config.getDir().exists() && !config.getDir().isDirectory())
 +      throw new IllegalArgumentException("Must pass in directory, " + config.getDir() + " is a file");
 +    
 +    if (config.getDir().exists() && config.getDir().list().length != 0)
 +      throw new IllegalArgumentException("Directory " + config.getDir() + " is not empty");
 +    
 +    this.config = config;
 +    
 +    libDir = new File(config.getDir(), "lib");
 +    libExtDir = new File(libDir, "ext");
 +    confDir = new File(config.getDir(), "conf");
 +    accumuloDir = new File(config.getDir(), "accumulo");
 +    zooKeeperDir = new File(config.getDir(), "zookeeper");
 +    logDir = new File(config.getDir(), "logs");
 +    walogDir = new File(config.getDir(), "walogs");
 +    
 +    confDir.mkdirs();
 +    accumuloDir.mkdirs();
 +    zooKeeperDir.mkdirs();
 +    logDir.mkdirs();
 +    walogDir.mkdirs();
 +    libDir.mkdirs();
 +    
 +    // Avoid the classloader yelling that the general.dynamic.classpaths value is invalid because
 +    // $ACCUMULO_HOME/lib/ext isn't defined.
 +    libExtDir.mkdirs();
 +    
 +    zooKeeperPort = PortUtils.getRandomFreePort();
 +    
 +    File siteFile = new File(confDir, "accumulo-site.xml");
 +    
 +    OutputStreamWriter fileWriter = new OutputStreamWriter(new FileOutputStream(siteFile), Constants.UTF8);
 +    fileWriter.append("<configuration>\n");
 +    
 +    HashMap<String,String> siteConfig = new HashMap<String,String>(config.getSiteConfig());
 +    
 +    appendProp(fileWriter, Property.INSTANCE_DFS_URI, "file:///", siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_DFS_DIR, accumuloDir.getAbsolutePath(), siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_ZK_HOST, "localhost:" + zooKeeperPort, siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_SECRET, INSTANCE_SECRET, siteConfig);
 +    appendProp(fileWriter, Property.TSERV_PORTSEARCH, "true", siteConfig);
 +    appendProp(fileWriter, Property.LOGGER_DIR, walogDir.getAbsolutePath(), siteConfig);
 +    appendProp(fileWriter, Property.TSERV_DATACACHE_SIZE, "10M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_INDEXCACHE_SIZE, "10M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_MAXMEM, "50M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_WALOG_MAX_SIZE, "100M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_NATIVEMAP_ENABLED, "false", siteConfig);
 +    appendProp(fileWriter, Property.TRACE_TOKEN_PROPERTY_PREFIX + ".password", config.getRootPassword(), siteConfig);
 +    appendProp(fileWriter, Property.GC_CYCLE_DELAY, "4s", siteConfig);
 +    appendProp(fileWriter, Property.GC_CYCLE_START, "0s", siteConfig);
 +    mergePropWithRandomPort(siteConfig, Property.MASTER_CLIENTPORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.TRACE_PORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.TSERV_CLIENTPORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.MONITOR_PORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.GC_PORT.getKey());
 +    
 +    // since there is a small amount of memory, check more frequently for majc... setting may not be needed in 1.5
 +    appendProp(fileWriter, Property.TSERV_MAJC_DELAY, "3", siteConfig);
 +    
 +    // ACCUMULO-1472 -- Use the classpath, not what might be installed on the system.
 +    // We have to set *something* here, otherwise the AccumuloClassLoader will default to pulling from 
 +    // environment variables (e.g. ACCUMULO_HOME, HADOOP_HOME/PREFIX) which will result in multiple copies
 +    // of artifacts on the classpath as they'll be provided by the invoking application
 +    appendProp(fileWriter, Property.GENERAL_CLASSPATHS, libDir.getAbsolutePath() + "/[^.].*.jar", siteConfig);
 +    appendProp(fileWriter, Property.GENERAL_DYNAMIC_CLASSPATHS, libExtDir.getAbsolutePath() + "/[^.].*.jar", siteConfig);
 +
 +    for (Entry<String,String> entry : siteConfig.entrySet())
 +      fileWriter.append("<property><name>" + entry.getKey() + "</name><value>" + entry.getValue() + "</value></property>\n");
 +    fileWriter.append("</configuration>\n");
 +    fileWriter.close();
 +    
 +    zooCfgFile = new File(confDir, "zoo.cfg");
 +    fileWriter = new OutputStreamWriter(new FileOutputStream(zooCfgFile), Constants.UTF8);
 +    
 +    // zookeeper uses Properties to read its config, so use that to write in order to properly escape things like Windows paths
 +    Properties zooCfg = new Properties();
 +    zooCfg.setProperty("tickTime", "1000");
 +    zooCfg.setProperty("initLimit", "10");
 +    zooCfg.setProperty("syncLimit", "5");
 +    zooCfg.setProperty("clientPort", zooKeeperPort + "");
 +    zooCfg.setProperty("maxClientCnxns", "100");
 +    zooCfg.setProperty("dataDir", zooKeeperDir.getAbsolutePath());
 +    zooCfg.store(fileWriter, null);
 +    
 +    fileWriter.close();
 +    
 +  }
 +  
 +  /**
 +   * Starts Accumulo and Zookeeper processes. Can only be called once.
 +   * 
 +   * @throws IOException
 +   * @throws InterruptedException
 +   * @throws IllegalStateException
 +   *           if already started
 +   */
 +  
 +  public void start() throws IOException, InterruptedException {
 +    if (zooKeeperProcess != null)
 +      throw new IllegalStateException("Already started");
 +    
 +    Runtime.getRuntime().addShutdownHook(new Thread() {
 +      @Override
 +      public void run() {
 +        try {
 +          MiniAccumuloCluster.this.stop();
 +        } catch (IOException e) {
 +          e.printStackTrace();
 +        } catch (InterruptedException e) {
 +          e.printStackTrace();
 +        }
 +      }
 +    });
 +    
 +    zooKeeperProcess = exec(Main.class, ZooKeeperServerMain.class.getName(), zooCfgFile.getAbsolutePath());
 +    
 +    // sleep a little bit to let zookeeper come up before calling init, seems to work better
 +    UtilWaitThread.sleep(250);
 +    
 +    Process initProcess = exec(Initialize.class, "--instance-name", INSTANCE_NAME, "--password", config.getRootPassword());
 +    int ret = initProcess.waitFor();
 +    if (ret != 0) {
 +      throw new RuntimeException("Initialize process returned " + ret + ". Check the logs in " + logDir + " for errors.");
 +    }
 +    
 +    tabletServerProcesses = new Process[config.getNumTservers()];
 +    for (int i = 0; i < config.getNumTservers(); i++) {
 +      tabletServerProcesses[i] = exec(TabletServer.class);
 +    }
 +    
 +    masterProcess = exec(Master.class);
 +    
 +    gcProcess = exec(SimpleGarbageCollector.class);
 +  }
 +  
 +  /**
 +   * @return Accumulo instance name
 +   */
 +  
 +  public String getInstanceName() {
 +    return INSTANCE_NAME;
 +  }
 +  
 +  /**
 +   * @return zookeeper connection string
 +   */
 +  
 +  public String getZooKeepers() {
 +    return "localhost:" + zooKeeperPort;
 +  }
 +  
 +  /**
 +   * Stops Accumulo and Zookeeper processes. If stop is not called, there is a shutdown hook that is setup to kill the processes. Howerver its probably best to
 +   * call stop in a finally block as soon as possible.
 +   * 
 +   * @throws IOException
 +   * @throws InterruptedException
 +   */
 +  
 +  public void stop() throws IOException, InterruptedException {
-     if (zooKeeperProcess != null)
++    if (zooKeeperProcess != null) {
 +      zooKeeperProcess.destroy();
-     if (masterProcess != null)
++      zooKeeperProcess.waitFor();
++    }
++    if (masterProcess != null) {
 +      masterProcess.destroy();
++      masterProcess.waitFor();
++    }
 +    if (tabletServerProcesses != null) {
 +      for (Process tserver : tabletServerProcesses) {
 +        tserver.destroy();
++        tserver.waitFor();
 +      }
 +    }
 +    
 +    for (LogWriter lw : logWriters)
 +      lw.flush();
 +
-     if (gcProcess != null)
++    if (gcProcess != null) {
 +      gcProcess.destroy();
++      gcProcess.waitFor();
++    }
 +  }
 +}


[04/10] git commit: ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()

Posted by el...@apache.org.
ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/0753a754
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/0753a754
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/0753a754

Branch: refs/heads/master
Commit: 0753a754b0a1751ab7f2bb4ddd922aa9dd492252
Parents: bbd6aeb
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:35:04 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:35:04 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java  | 17 +++++++++++++----
 1 file changed, 13 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/0753a754/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --git a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index cbed375..848e14f 100644
--- a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@ -362,22 +362,31 @@ public class MiniAccumuloCluster {
    */
   
   public void stop() throws IOException, InterruptedException {
-    if (zooKeeperProcess != null)
+    if (zooKeeperProcess != null) {
       zooKeeperProcess.destroy();
-    if (loggerProcess != null)
+      zooKeeperProcess.waitFor();
+    }
+    if (loggerProcess != null) {
       loggerProcess.destroy();
-    if (masterProcess != null)
+      loggerProcess.waitFor();
+    }
+    if (masterProcess != null) {
       masterProcess.destroy();
+      masterProcess.waitFor();
+    }
     if (tabletServerProcesses != null) {
       for (Process tserver : tabletServerProcesses) {
         tserver.destroy();
+        tserver.waitFor();
       }
     }
     
     for (LogWriter lw : logWriters)
       lw.flush();
 
-    if (gcProcess != null)
+    if (gcProcess != null) {
       gcProcess.destroy();
+      gcProcess.waitFor();
+    }
   }
 }


[09/10] git commit: Merge branch '1.5.2-SNAPSHOT' into 1.6.0-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.5.2-SNAPSHOT' into 1.6.0-SNAPSHOT

Conflicts:
	minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/44b13c12
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/44b13c12
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/44b13c12

Branch: refs/heads/master
Commit: 44b13c12e5555b9da6165affe038f825a6b17fa2
Parents: aefa52e 63d5e55
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 17:48:51 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 17:48:51 2014 -0400

----------------------------------------------------------------------
 .../minicluster/impl/MiniAccumuloClusterImpl.java       | 12 +++++++++++-
 1 file changed, 11 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/44b13c12/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
----------------------------------------------------------------------
diff --cc minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
index 8a9736d,0000000..009988e
mode 100644,000000..100644
--- a/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
+++ b/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
@@@ -1,649 -1,0 +1,659 @@@
 +/*
 + * Licensed to the Apache Software Foundation (ASF) under one or more
 + * contributor license agreements.  See the NOTICE file distributed with
 + * this work for additional information regarding copyright ownership.
 + * The ASF licenses this file to You under the Apache License, Version 2.0
 + * (the "License"); you may not use this file except in compliance with
 + * the License.  You may obtain a copy of the License at
 + *
 + *     http://www.apache.org/licenses/LICENSE-2.0
 + *
 + * Unless required by applicable law or agreed to in writing, software
 + * distributed under the License is distributed on an "AS IS" BASIS,
 + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
 + * See the License for the specific language governing permissions and
 + * limitations under the License.
 + */
 +package org.apache.accumulo.minicluster.impl;
 +
 +import java.io.BufferedReader;
 +import java.io.BufferedWriter;
 +import java.io.File;
 +import java.io.FileFilter;
 +import java.io.FileWriter;
 +import java.io.IOException;
 +import java.io.InputStream;
 +import java.io.InputStreamReader;
 +import java.net.InetSocketAddress;
 +import java.net.Socket;
 +import java.net.URI;
 +import java.net.URISyntaxException;
 +import java.net.URL;
 +import java.net.URLClassLoader;
 +import java.util.ArrayList;
 +import java.util.Arrays;
 +import java.util.Collection;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.HashSet;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Properties;
 +import java.util.Set;
 +
 +import org.apache.accumulo.core.client.AccumuloException;
 +import org.apache.accumulo.core.client.AccumuloSecurityException;
 +import org.apache.accumulo.core.client.ClientConfiguration;
 +import org.apache.accumulo.core.client.Connector;
 +import org.apache.accumulo.core.client.Instance;
 +import org.apache.accumulo.core.client.ZooKeeperInstance;
 +import org.apache.accumulo.core.client.security.tokens.PasswordToken;
 +import org.apache.accumulo.core.conf.Property;
 +import org.apache.accumulo.core.master.thrift.MasterGoalState;
 +import org.apache.accumulo.core.util.Daemon;
 +import org.apache.accumulo.core.util.Pair;
 +import org.apache.accumulo.core.util.StringUtil;
 +import org.apache.accumulo.core.util.UtilWaitThread;
 +import org.apache.accumulo.gc.SimpleGarbageCollector;
 +import org.apache.accumulo.master.Master;
 +import org.apache.accumulo.master.state.SetGoalState;
 +import org.apache.accumulo.minicluster.ServerType;
 +import org.apache.accumulo.server.init.Initialize;
 +import org.apache.accumulo.server.util.PortUtils;
 +import org.apache.accumulo.server.util.time.SimpleTimer;
 +import org.apache.accumulo.start.Main;
 +import org.apache.accumulo.start.classloader.vfs.MiniDFSUtil;
 +import org.apache.accumulo.tserver.TabletServer;
 +import org.apache.commons.configuration.MapConfiguration;
 +import org.apache.commons.vfs2.FileObject;
 +import org.apache.commons.vfs2.impl.VFSClassLoader;
 +import org.apache.hadoop.conf.Configuration;
 +import org.apache.hadoop.fs.FileSystem;
 +import org.apache.hadoop.hdfs.DFSConfigKeys;
 +import org.apache.hadoop.hdfs.MiniDFSCluster;
 +import org.apache.zookeeper.server.ZooKeeperServerMain;
 +
 +import com.google.common.base.Predicate;
 +import com.google.common.collect.Maps;
 +
 +/**
 + * A utility class that will create Zookeeper and Accumulo processes that write all of their data to a single local directory. This class makes it easy to test
 + * code against a real Accumulo instance. Its much more accurate for testing than {@link org.apache.accumulo.core.client.mock.MockAccumulo}, but much slower.
 + * 
 + * @since 1.5.0
 + */
 +public class MiniAccumuloClusterImpl {
 +
 +  public static class LogWriter extends Daemon {
 +    private BufferedReader in;
 +    private BufferedWriter out;
 +
 +    public LogWriter(InputStream stream, File logFile) throws IOException {
 +      this.in = new BufferedReader(new InputStreamReader(stream));
 +      out = new BufferedWriter(new FileWriter(logFile));
 +
 +      SimpleTimer.getInstance().schedule(new Runnable() {
 +        @Override
 +        public void run() {
 +          try {
 +            flush();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      }, 1000, 1000);
 +    }
 +
 +    public synchronized void flush() throws IOException {
 +      if (out != null)
 +        out.flush();
 +    }
 +
 +    @Override
 +    public void run() {
 +      String line;
 +
 +      try {
 +        while ((line = in.readLine()) != null) {
 +          out.append(line);
 +          out.append("\n");
 +        }
 +
 +        synchronized (this) {
 +          out.close();
 +          out = null;
 +          in.close();
 +        }
 +
 +      } catch (IOException e) {}
 +    }
 +  }
 +
 +  private static final long ZOOKEEPER_STARTUP_WAIT = 20*1000;
 +
 +  private boolean initialized = false;
 +  private Process zooKeeperProcess = null;
 +  private Process masterProcess = null;
 +  private Process gcProcess = null;
 +  private List<Process> tabletServerProcesses = Collections.synchronizedList(new ArrayList<Process>());
 +
 +  private Set<Pair<ServerType,Integer>> debugPorts = new HashSet<Pair<ServerType,Integer>>();
 +
 +  private File zooCfgFile;
 +  private String dfsUri;
 +
 +  public List<LogWriter> getLogWriters() {
 +    return logWriters;
 +  }
 +
 +  private List<LogWriter> logWriters = new ArrayList<MiniAccumuloClusterImpl.LogWriter>();
 +
 +  private MiniAccumuloConfigImpl config;
 +  private MiniDFSCluster miniDFS = null;
 +  private List<Process> cleanup = new ArrayList<Process>();
 +
 +  public Process exec(Class<?> clazz, String... args) throws IOException {
 +    return exec(clazz, null, args);
 +  }
 +
 +  public Process exec(Class<?> clazz, List<String> jvmArgs, String... args) throws IOException {
 +    ArrayList<String> jvmArgs2 = new ArrayList<String>(1 + (jvmArgs == null ? 0 : jvmArgs.size()));
 +    jvmArgs2.add("-Xmx" + config.getDefaultMemory());
 +    if (jvmArgs != null)
 +      jvmArgs2.addAll(jvmArgs);
 +    Process proc = _exec(clazz, jvmArgs2, args);
 +    cleanup.add(proc);
 +    return proc;
 +  }
 +
 +  private boolean containsSiteFile(File f) {
 +    return f.isDirectory() && f.listFiles(new FileFilter() {
 +
 +      @Override
 +      public boolean accept(File pathname) {
 +        return pathname.getName().endsWith("site.xml");
 +      }
 +    }).length > 0;
 +  }
 +
 +  private void append(StringBuilder classpathBuilder, URL url) throws URISyntaxException {
 +    File file = new File(url.toURI());
 +    // do not include dirs containing hadoop or accumulo site files
 +    if (!containsSiteFile(file))
 +      classpathBuilder.append(File.pathSeparator).append(file.getAbsolutePath());
 +  }
 +
 +  private String getClasspath() throws IOException {
 +
 +    try {
 +      ArrayList<ClassLoader> classloaders = new ArrayList<ClassLoader>();
 +
 +      ClassLoader cl = this.getClass().getClassLoader();
 +
 +      while (cl != null) {
 +        classloaders.add(cl);
 +        cl = cl.getParent();
 +      }
 +
 +      Collections.reverse(classloaders);
 +
 +      StringBuilder classpathBuilder = new StringBuilder();
 +      classpathBuilder.append(config.getConfDir().getAbsolutePath());
 +
 +      if (config.getClasspathItems() == null) {
 +
 +        // assume 0 is the system classloader and skip it
 +        for (int i = 1; i < classloaders.size(); i++) {
 +          ClassLoader classLoader = classloaders.get(i);
 +
 +          if (classLoader instanceof URLClassLoader) {
 +
 +            URLClassLoader ucl = (URLClassLoader) classLoader;
 +
 +            for (URL u : ucl.getURLs()) {
 +              append(classpathBuilder, u);
 +            }
 +
 +          } else if (classLoader instanceof VFSClassLoader) {
 +
 +            VFSClassLoader vcl = (VFSClassLoader) classLoader;
 +            for (FileObject f : vcl.getFileObjects()) {
 +              append(classpathBuilder, f.getURL());
 +            }
 +          } else {
 +            throw new IllegalArgumentException("Unknown classloader type : " + classLoader.getClass().getName());
 +          }
 +        }
 +      } else {
 +        for (String s : config.getClasspathItems())
 +          classpathBuilder.append(File.pathSeparator).append(s);
 +      }
 +
 +      return classpathBuilder.toString();
 +
 +    } catch (URISyntaxException e) {
 +      throw new IOException(e);
 +    }
 +  }
 +
 +  private Process _exec(Class<?> clazz, List<String> extraJvmOpts, String... args) throws IOException {
 +    String javaHome = System.getProperty("java.home");
 +    String javaBin = javaHome + File.separator + "bin" + File.separator + "java";
 +    String classpath = getClasspath();
 +
 +    String className = clazz.getName();
 +
 +    ArrayList<String> argList = new ArrayList<String>();
 +    argList.addAll(Arrays.asList(javaBin, "-Dproc=" + clazz.getSimpleName(), "-cp", classpath));
 +    argList.addAll(extraJvmOpts);
 +    for (Entry<String,String> sysProp : config.getSystemProperties().entrySet()) {
 +      argList.add(String.format("-D%s=%s", sysProp.getKey(), sysProp.getValue()));
 +    }
 +    argList.addAll(Arrays.asList("-XX:+UseConcMarkSweepGC", "-XX:CMSInitiatingOccupancyFraction=75", "-Dapple.awt.UIElement=true", Main.class.getName(), className));
 +    argList.addAll(Arrays.asList(args));
 +
 +    ProcessBuilder builder = new ProcessBuilder(argList);
 +
 +    builder.environment().put("ACCUMULO_HOME", config.getDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_LOG_DIR", config.getLogDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_CLIENT_CONF_PATH", config.getClientConfFile().getAbsolutePath());
 +    String ldLibraryPath = StringUtil.join(Arrays.asList(config.getNativeLibPaths()), File.pathSeparator);
 +    builder.environment().put("LD_LIBRARY_PATH", ldLibraryPath);
 +    builder.environment().put("DYLD_LIBRARY_PATH", ldLibraryPath);
 +
 +    // if we're running under accumulo.start, we forward these env vars
 +    String env = System.getenv("HADOOP_PREFIX");
 +    if (env != null)
 +      builder.environment().put("HADOOP_PREFIX", env);
 +    env = System.getenv("ZOOKEEPER_HOME");
 +    if (env != null)
 +      builder.environment().put("ZOOKEEPER_HOME", env);
 +    builder.environment().put("ACCUMULO_CONF_DIR", config.getConfDir().getAbsolutePath());
 +    // hadoop-2.2 puts error messages in the logs if this is not set
 +    builder.environment().put("HADOOP_HOME", config.getDir().getAbsolutePath());
 +
 +    Process process = builder.start();
 +
 +    LogWriter lw;
 +    lw = new LogWriter(process.getErrorStream(), new File(config.getLogDir(), clazz.getSimpleName() + "_" + process.hashCode() + ".err"));
 +    logWriters.add(lw);
 +    lw.start();
 +    lw = new LogWriter(process.getInputStream(), new File(config.getLogDir(), clazz.getSimpleName() + "_" + process.hashCode() + ".out"));
 +    logWriters.add(lw);
 +    lw.start();
 +
 +    return process;
 +  }
 +
 +  private Process _exec(Class<?> clazz, ServerType serverType, String... args) throws IOException {
 +
 +    List<String> jvmOpts = new ArrayList<String>();
 +    jvmOpts.add("-Xmx" + config.getMemory(serverType));
 +
 +    if (config.isJDWPEnabled()) {
 +      Integer port = PortUtils.getRandomFreePort();
 +      jvmOpts.addAll(buildRemoteDebugParams(port));
 +      debugPorts.add(new Pair<ServerType,Integer>(serverType, port));
 +    }
 +    return _exec(clazz, jvmOpts, args);
 +  }
 +
 +  /**
 +   * 
 +   * @param dir
 +   *          An empty or nonexistant temp directoy that Accumulo and Zookeeper can store data in. Creating the directory is left to the user. Java 7, Guava,
 +   *          and Junit provide methods for creating temporary directories.
 +   * @param rootPassword
 +   *          Initial root password for instance.
 +   */
 +  public MiniAccumuloClusterImpl(File dir, String rootPassword) throws IOException {
 +    this(new MiniAccumuloConfigImpl(dir, rootPassword));
 +  }
 +
 +  /**
 +   * @param config
 +   *          initial configuration
 +   */
 +  @SuppressWarnings("deprecation")
 +  public MiniAccumuloClusterImpl(MiniAccumuloConfigImpl config) throws IOException {
 +
 +    this.config = config.initialize();
 +
 +    config.getConfDir().mkdirs();
 +    config.getAccumuloDir().mkdirs();
 +    config.getZooKeeperDir().mkdirs();
 +    config.getLogDir().mkdirs();
 +    config.getWalogDir().mkdirs();
 +    config.getLibDir().mkdirs();
 +    config.getLibExtDir().mkdirs();
 +
 +    if (config.useMiniDFS()) {
 +      File nn = new File(config.getAccumuloDir(), "nn");
 +      nn.mkdirs();
 +      File dn = new File(config.getAccumuloDir(), "dn");
 +      dn.mkdirs();
 +      File dfs = new File(config.getAccumuloDir(), "dfs");
 +      dfs.mkdirs();
 +      Configuration conf = new Configuration();
 +      conf.set(DFSConfigKeys.DFS_NAMENODE_NAME_DIR_KEY, nn.getAbsolutePath());
 +      conf.set(DFSConfigKeys.DFS_DATANODE_DATA_DIR_KEY, dn.getAbsolutePath());
 +      conf.set(DFSConfigKeys.DFS_REPLICATION_KEY, "1");
 +      conf.set("dfs.support.append", "true");
 +      conf.set("dfs.datanode.synconclose", "true");
 +      conf.set("dfs.datanode.data.dir.perm", MiniDFSUtil.computeDatanodeDirectoryPermission());
 +      String oldTestBuildData = System.setProperty("test.build.data", dfs.getAbsolutePath());
 +      miniDFS = new MiniDFSCluster(conf, 1, true, null);
 +      if (oldTestBuildData == null)
 +        System.clearProperty("test.build.data");
 +      else
 +        System.setProperty("test.build.data", oldTestBuildData);
 +      miniDFS.waitClusterUp();
 +      InetSocketAddress dfsAddress = miniDFS.getNameNode().getNameNodeAddress();
 +      dfsUri = "hdfs://" + dfsAddress.getHostName() + ":" + dfsAddress.getPort();
 +      File coreFile = new File(config.getConfDir(), "core-site.xml");
 +      writeConfig(coreFile, Collections.singletonMap("fs.default.name", dfsUri).entrySet());
 +      File hdfsFile = new File(config.getConfDir(), "hdfs-site.xml");
 +      writeConfig(hdfsFile, conf);
 +
 +      Map<String,String> siteConfig = config.getSiteConfig();
 +      siteConfig.put(Property.INSTANCE_DFS_URI.getKey(), dfsUri);
 +      siteConfig.put(Property.INSTANCE_DFS_DIR.getKey(), "/accumulo");
 +      config.setSiteConfig(siteConfig);
 +    } else {
 +      dfsUri = "file://";
 +    }
 +
 +    File clientConfFile = config.getClientConfFile();
 +    // Write only the properties that correspond to ClientConfiguration properties
 +    writeConfigProperties(clientConfFile, Maps.filterEntries(config.getSiteConfig(), new Predicate<Entry<String,String>>() {
 +      @Override
 +      public boolean apply(Entry<String,String> v) {
 +        return ClientConfiguration.ClientProperty.getPropertyByKey(v.getKey()) != null;
 +      }
 +    }));
 +
 +    File siteFile = new File(config.getConfDir(), "accumulo-site.xml");
 +    writeConfig(siteFile, config.getSiteConfig().entrySet());
 +
 +    zooCfgFile = new File(config.getConfDir(), "zoo.cfg");
 +    FileWriter fileWriter = new FileWriter(zooCfgFile);
 +
 +    // zookeeper uses Properties to read its config, so use that to write in order to properly escape things like Windows paths
 +    Properties zooCfg = new Properties();
 +    zooCfg.setProperty("tickTime", "2000");
 +    zooCfg.setProperty("initLimit", "10");
 +    zooCfg.setProperty("syncLimit", "5");
 +    zooCfg.setProperty("clientPort", config.getZooKeeperPort() + "");
 +    zooCfg.setProperty("maxClientCnxns", "1000");
 +    zooCfg.setProperty("dataDir", config.getZooKeeperDir().getAbsolutePath());
 +    zooCfg.store(fileWriter, null);
 +
 +    fileWriter.close();
 +  }
 +
 +  private void writeConfig(File file, Iterable<Map.Entry<String,String>> settings) throws IOException {
 +    FileWriter fileWriter = new FileWriter(file);
 +    fileWriter.append("<configuration>\n");
 +
 +    for (Entry<String,String> entry : settings) {
 +      String value = entry.getValue().replace("&", "&amp;").replace("<", "&lt;").replace(">", "&gt;");
 +      fileWriter.append("<property><name>" + entry.getKey() + "</name><value>" + value + "</value></property>\n");
 +    }
 +    fileWriter.append("</configuration>\n");
 +    fileWriter.close();
 +  }
 +
 +  private void writeConfigProperties(File file, Map<String,String> settings) throws IOException {
 +    FileWriter fileWriter = new FileWriter(file);
 +
 +    for (Entry<String,String> entry : settings.entrySet())
 +      fileWriter.append(entry.getKey() + "=" + entry.getValue() + "\n");
 +    fileWriter.close();
 +  }
 +
 +  /**
 +   * Starts Accumulo and Zookeeper processes. Can only be called once.
 +   * 
 +   * @throws IllegalStateException
 +   *           if already started
 +   */
 +  public void start() throws IOException, InterruptedException {
 +
 +    if (!initialized) {
 +
 +      Runtime.getRuntime().addShutdownHook(new Thread() {
 +        @Override
 +        public void run() {
 +          try {
 +            MiniAccumuloClusterImpl.this.stop();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          } catch (InterruptedException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      });
 +    }
 +
 +    if (zooKeeperProcess == null) {
 +      zooKeeperProcess = _exec(ZooKeeperServerMain.class, ServerType.ZOOKEEPER, zooCfgFile.getAbsolutePath());
 +    }
 +
 +    if (!initialized) {
 +      // sleep a little bit to let zookeeper come up before calling init, seems to work better
 +      long startTime = System.currentTimeMillis();
 +      while (true) {
 +        Socket s = null;
 +        try {
 +          s = new Socket("localhost", config.getZooKeeperPort());
 +          s.getOutputStream().write("ruok\n".getBytes());
 +          s.getOutputStream().flush();
 +          byte buffer[] = new byte[100];
 +          int n = s.getInputStream().read(buffer);
 +          if (n >= 4 && new String(buffer, 0, 4).equals("imok"))
 +            break;
 +        } catch (Exception e) {
 +          if (System.currentTimeMillis() - startTime >= ZOOKEEPER_STARTUP_WAIT) {
 +            throw new RuntimeException("Zookeeper did not start within " + (ZOOKEEPER_STARTUP_WAIT/1000) + " seconds. Check the logs in " + config.getLogDir() + " for errors.  Last exception: " + e);
 +          }
 +          UtilWaitThread.sleep(250);
 +        } finally {
 +          if (s != null)
 +            s.close();
 +        }
 +      }
 +      Process initProcess = exec(Initialize.class, "--instance-name", config.getInstanceName(), "--password", config.getRootPassword());
 +      int ret = initProcess.waitFor();
 +      if (ret != 0) {
 +        throw new RuntimeException("Initialize process returned " + ret + ". Check the logs in " + config.getLogDir() + " for errors.");
 +      }
 +      initialized = true;
 +    }
 +    synchronized (tabletServerProcesses) {
 +      for (int i = tabletServerProcesses.size(); i < config.getNumTservers(); i++) {
 +        tabletServerProcesses.add(_exec(TabletServer.class, ServerType.TABLET_SERVER));
 +      }
 +    }
 +    int ret = 0;
 +    for (int i = 0; i < 5; i++) {
 +      ret = exec(Main.class, SetGoalState.class.getName(), MasterGoalState.NORMAL.toString()).waitFor();
 +      if (ret == 0)
 +        break;
 +      UtilWaitThread.sleep(1000);
 +    }
 +    if (ret != 0) {
 +      throw new RuntimeException("Could not set master goal state, process returned " + ret + ". Check the logs in " + config.getLogDir() + " for errors.");
 +    }
 +    if (masterProcess == null) {
 +      masterProcess = _exec(Master.class, ServerType.MASTER);
 +    }
 +
 +    if (gcProcess == null) {
 +      gcProcess = _exec(SimpleGarbageCollector.class, ServerType.GARBAGE_COLLECTOR);
 +    }
 +  }
 +
 +  private List<String> buildRemoteDebugParams(int port) {
 +    return Arrays.asList(new String[] {"-Xdebug", String.format("-Xrunjdwp:transport=dt_socket,server=y,suspend=n,address=%d", port)});
 +  }
 +
 +  /**
 +   * @return generated remote debug ports if in debug mode.
 +   * @since 1.6.0
 +   */
 +  public Set<Pair<ServerType,Integer>> getDebugPorts() {
 +    return debugPorts;
 +  }
 +
 +  List<ProcessReference> references(Process... procs) {
 +    List<ProcessReference> result = new ArrayList<ProcessReference>();
 +    for (Process proc : procs) {
 +      result.add(new ProcessReference(proc));
 +    }
 +    return result;
 +  }
 +
 +  public Map<ServerType,Collection<ProcessReference>> getProcesses() {
 +    Map<ServerType,Collection<ProcessReference>> result = new HashMap<ServerType,Collection<ProcessReference>>();
 +    result.put(ServerType.MASTER, references(masterProcess));
 +    result.put(ServerType.TABLET_SERVER, references(tabletServerProcesses.toArray(new Process[0])));
 +    result.put(ServerType.ZOOKEEPER, references(zooKeeperProcess));
 +    if (null != gcProcess) {
 +      result.put(ServerType.GARBAGE_COLLECTOR, references(gcProcess));
 +    }
 +    return result;
 +  }
 +
 +  public void killProcess(ServerType type, ProcessReference proc) throws ProcessNotFoundException, InterruptedException {
 +    boolean found = false;
 +    switch (type) {
 +      case MASTER:
 +        if (proc.equals(masterProcess)) {
 +          masterProcess.destroy();
++          masterProcess.waitFor();
 +          masterProcess = null;
 +          found = true;
 +        }
 +        break;
 +      case TABLET_SERVER:
 +        synchronized (tabletServerProcesses) {
 +          for (Process tserver : tabletServerProcesses) {
 +            if (proc.equals(tserver)) {
 +              tabletServerProcesses.remove(tserver);
 +              tserver.destroy();
++              tserver.waitFor();
 +              found = true;
 +              break;
 +            }
 +          }
 +        }
 +        break;
 +      case ZOOKEEPER:
 +        if (proc.equals(zooKeeperProcess)) {
 +          zooKeeperProcess.destroy();
++          zooKeeperProcess.waitFor();
 +          zooKeeperProcess = null;
 +          found = true;
 +        }
 +        break;
 +      case GARBAGE_COLLECTOR:
 +        if (proc.equals(gcProcess)) {
 +          gcProcess.destroy();
++          gcProcess.waitFor();
 +          gcProcess = null;
 +          found = true;
 +        }
 +        break;
 +    }
 +    if (!found)
 +      throw new ProcessNotFoundException();
 +  }
 +
 +  /**
 +   * @return Accumulo instance name
 +   */
 +  public String getInstanceName() {
 +    return config.getInstanceName();
 +  }
 +
 +  /**
 +   * @return zookeeper connection string
 +   */
 +  public String getZooKeepers() {
 +    return config.getZooKeepers();
 +  }
 +
 +  /**
 +   * Stops Accumulo and Zookeeper processes. If stop is not called, there is a shutdown hook that is setup to kill the processes. However its probably best to
 +   * call stop in a finally block as soon as possible.
 +   */
 +  public void stop() throws IOException, InterruptedException {
 +    for (LogWriter lw : logWriters) {
 +      lw.flush();
 +    }
 +
 +    if (zooKeeperProcess != null) {
 +      zooKeeperProcess.destroy();
++      zooKeeperProcess.waitFor();
 +    }
 +    if (masterProcess != null) {
 +      masterProcess.destroy();
++      masterProcess.waitFor();
 +    }
 +    if (tabletServerProcesses != null) {
 +      synchronized (tabletServerProcesses) {
 +        for (Process tserver : tabletServerProcesses) {
 +          tserver.destroy();
++          tserver.waitFor();
 +        }
 +      }
 +    }
 +    if (gcProcess != null) {
 +      gcProcess.destroy();
++      gcProcess.waitFor();
 +    }
 +
 +    zooKeeperProcess = null;
 +    masterProcess = null;
 +    gcProcess = null;
 +    tabletServerProcesses.clear();
 +    if (config.useMiniDFS() && miniDFS != null)
 +      miniDFS.shutdown();
-     for (Process p : cleanup)
++    for (Process p : cleanup) {
 +      p.destroy();
++      p.waitFor();
++    }
 +    miniDFS = null;
 +  }
 +
 +  /**
 +   * @since 1.6.0
 +   */
 +  public MiniAccumuloConfigImpl getConfig() {
 +    return config;
 +  }
 +
 +  /**
 +   * Utility method to get a connector to the MAC.
 +   * 
 +   * @since 1.6.0
 +   */
 +  public Connector getConnector(String user, String passwd) throws AccumuloException, AccumuloSecurityException {
 +    Instance instance = new ZooKeeperInstance(getClientConfig());
 +    return instance.getConnector(user, new PasswordToken(passwd));
 +  }
 +
 +  public ClientConfiguration getClientConfig() {
 +    return new ClientConfiguration(Arrays.asList(new MapConfiguration(config.getSiteConfig()))).withInstance(this.getInstanceName()).withZkHosts(
 +        this.getZooKeepers());
 +  }
 +
 +  public FileSystem getFileSystem() {
 +    try {
 +      return FileSystem.get(new URI(dfsUri), new Configuration());
 +    } catch (Exception e) {
 +      throw new RuntimeException(e);
 +    }
 +  }
 +}


[10/10] git commit: Merge branch '1.6.0-SNAPSHOT'

Posted by el...@apache.org.
Merge branch '1.6.0-SNAPSHOT'


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/394fe061
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/394fe061
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/394fe061

Branch: refs/heads/master
Commit: 394fe061fb896e740b0932a9736c0fcc16caee95
Parents: b644eda 44b13c1
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 17:48:57 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 17:48:57 2014 -0400

----------------------------------------------------------------------
 .../minicluster/impl/MiniAccumuloClusterImpl.java       | 12 +++++++++++-
 1 file changed, 11 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/394fe061/minicluster/src/main/java/org/apache/accumulo/minicluster/impl/MiniAccumuloClusterImpl.java
----------------------------------------------------------------------


[02/10] git commit: ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()

Posted by el...@apache.org.
ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/0753a754
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/0753a754
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/0753a754

Branch: refs/heads/1.5.2-SNAPSHOT
Commit: 0753a754b0a1751ab7f2bb4ddd922aa9dd492252
Parents: bbd6aeb
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:35:04 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:35:04 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java  | 17 +++++++++++++----
 1 file changed, 13 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/0753a754/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --git a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index cbed375..848e14f 100644
--- a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@ -362,22 +362,31 @@ public class MiniAccumuloCluster {
    */
   
   public void stop() throws IOException, InterruptedException {
-    if (zooKeeperProcess != null)
+    if (zooKeeperProcess != null) {
       zooKeeperProcess.destroy();
-    if (loggerProcess != null)
+      zooKeeperProcess.waitFor();
+    }
+    if (loggerProcess != null) {
       loggerProcess.destroy();
-    if (masterProcess != null)
+      loggerProcess.waitFor();
+    }
+    if (masterProcess != null) {
       masterProcess.destroy();
+      masterProcess.waitFor();
+    }
     if (tabletServerProcesses != null) {
       for (Process tserver : tabletServerProcesses) {
         tserver.destroy();
+        tserver.waitFor();
       }
     }
     
     for (LogWriter lw : logWriters)
       lw.flush();
 
-    if (gcProcess != null)
+    if (gcProcess != null) {
       gcProcess.destroy();
+      gcProcess.waitFor();
+    }
   }
 }


[07/10] git commit: Merge branch '1.4.5-SNAPSHOT' into 1.5.2-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.4.5-SNAPSHOT' into 1.5.2-SNAPSHOT

Conflicts:
	minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/63d5e55a
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/63d5e55a
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/63d5e55a

Branch: refs/heads/master
Commit: 63d5e55a0b03910246b9b21efecfde5ac5e709f0
Parents: 4b7a011 0753a75
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:50:59 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:50:59 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java      | 13 ++++++++++---
 1 file changed, 10 insertions(+), 3 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/63d5e55a/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --cc minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index 4018a33,0000000..a366c16
mode 100644,000000..100644
--- a/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@@ -1,385 -1,0 +1,392 @@@
 +/*
 + * Licensed to the Apache Software Foundation (ASF) under one or more
 + * contributor license agreements.  See the NOTICE file distributed with
 + * this work for additional information regarding copyright ownership.
 + * The ASF licenses this file to You under the Apache License, Version 2.0
 + * (the "License"); you may not use this file except in compliance with
 + * the License.  You may obtain a copy of the License at
 + *
 + *     http://www.apache.org/licenses/LICENSE-2.0
 + *
 + * Unless required by applicable law or agreed to in writing, software
 + * distributed under the License is distributed on an "AS IS" BASIS,
 + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
 + * See the License for the specific language governing permissions and
 + * limitations under the License.
 + */
 +package org.apache.accumulo.minicluster;
 +
 +import java.io.BufferedReader;
 +import java.io.BufferedWriter;
 +import java.io.File;
 +import java.io.FileOutputStream;
 +import java.io.IOException;
 +import java.io.InputStream;
 +import java.io.InputStreamReader;
 +import java.io.OutputStreamWriter;
 +import java.io.Writer;
 +import java.util.ArrayList;
 +import java.util.Arrays;
 +import java.util.HashMap;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Properties;
 +
 +import org.apache.accumulo.core.Constants;
 +import org.apache.accumulo.core.conf.Property;
 +import org.apache.accumulo.core.util.UtilWaitThread;
 +import org.apache.accumulo.server.gc.SimpleGarbageCollector;
 +import org.apache.accumulo.server.master.Master;
 +import org.apache.accumulo.server.tabletserver.TabletServer;
 +import org.apache.accumulo.server.util.Initialize;
 +import org.apache.accumulo.server.util.PortUtils;
 +import org.apache.accumulo.server.util.time.SimpleTimer;
 +import org.apache.accumulo.start.Main;
 +import org.apache.zookeeper.server.ZooKeeperServerMain;
 +
 +/**
 + * A utility class that will create Zookeeper and Accumulo processes that write all of their data to a single local directory. This class makes it easy to test
 + * code against a real Accumulo instance. Its much more accurate for testing than MockAccumulo, but much slower than MockAccumulo.
 + * 
 + * @since 1.5.0
 + */
 +public class MiniAccumuloCluster {
 +  
 +  private static final String INSTANCE_SECRET = "DONTTELL";
 +  private static final String INSTANCE_NAME = "miniInstance";
 +  
 +  private static class LogWriter extends Thread {
 +    private BufferedReader in;
 +    private BufferedWriter out;
 +    
 +    /**
 +     * @throws IOException
 +     */
 +    public LogWriter(InputStream stream, File logFile) throws IOException {
 +      this.setDaemon(true);
 +      this.in = new BufferedReader(new InputStreamReader(stream, Constants.UTF8));
 +      out = new BufferedWriter(new OutputStreamWriter(new FileOutputStream(logFile), Constants.UTF8));
 +      
 +      SimpleTimer.getInstance().schedule(new Runnable() {
 +        @Override
 +        public void run() {
 +          try {
 +            flush();
 +          } catch (IOException e) {
 +            e.printStackTrace();
 +          }
 +        }
 +      }, 1000, 1000);
 +    }
 +    
 +    public synchronized void flush() throws IOException {
 +      if (out != null)
 +        out.flush();
 +    }
 +    
 +    @Override
 +    public void run() {
 +      String line;
 +      
 +      try {
 +        while ((line = in.readLine()) != null) {
 +          out.append(line);
 +          out.append("\n");
 +        }
 +        
 +        synchronized (this) {
 +          out.close();
 +          out = null;
 +          in.close();
 +        }
 +        
 +      } catch (IOException e) {}
 +    }
 +  }
 +  
 +  private File libDir;
 +  private File libExtDir;
 +  private File confDir;
 +  private File zooKeeperDir;
 +  private File accumuloDir;
 +  private File zooCfgFile;
 +  private File logDir;
 +  private File walogDir;
 +  
 +  private Process zooKeeperProcess;
 +  private Process masterProcess;
 +  private Process gcProcess;
 +  
 +  private int zooKeeperPort;
 +  
 +  private List<LogWriter> logWriters = new ArrayList<MiniAccumuloCluster.LogWriter>();
 +  
 +  private MiniAccumuloConfig config;
 +  private Process[] tabletServerProcesses;
 +  
 +  private Process exec(Class<? extends Object> clazz, String... args) throws IOException {
 +    String javaHome = System.getProperty("java.home");
 +    String javaBin = javaHome + File.separator + "bin" + File.separator + "java";
 +    String classpath = System.getProperty("java.class.path");
 +    
 +    classpath = confDir.getAbsolutePath() + File.pathSeparator + classpath;
 +    
 +    String className = clazz.getCanonicalName();
 +    
 +    ArrayList<String> argList = new ArrayList<String>();
 +    
 +    argList.addAll(Arrays.asList(javaBin, "-cp", classpath, "-Xmx128m", "-XX:+UseConcMarkSweepGC", "-XX:CMSInitiatingOccupancyFraction=75",
 +        "-Dapple.awt.UIElement=true", Main.class.getName(), className));
 +    
 +    argList.addAll(Arrays.asList(args));
 +    
 +    ProcessBuilder builder = new ProcessBuilder(argList);
 +    
 +    builder.environment().put("ACCUMULO_HOME", config.getDir().getAbsolutePath());
 +    builder.environment().put("ACCUMULO_LOG_DIR", logDir.getAbsolutePath());
 +    
 +    // if we're running under accumulo.start, we forward these env vars
 +    String env = System.getenv("HADOOP_PREFIX");
 +    if (env != null)
 +      builder.environment().put("HADOOP_PREFIX", env);
 +    env = System.getenv("ZOOKEEPER_HOME");
 +    if (env != null)
 +      builder.environment().put("ZOOKEEPER_HOME", env);
 +    
 +    Process process = builder.start();
 +    
 +    LogWriter lw;
 +    lw = new LogWriter(process.getErrorStream(), new File(logDir, clazz.getSimpleName() + "_" + process.hashCode() + ".err"));
 +    logWriters.add(lw);
 +    lw.start();
 +    lw = new LogWriter(process.getInputStream(), new File(logDir, clazz.getSimpleName() + "_" + process.hashCode() + ".out"));
 +    logWriters.add(lw);
 +    lw.start();
 +    
 +    return process;
 +  }
 +  
 +  private void appendProp(Writer fileWriter, Property key, String value, Map<String,String> siteConfig) throws IOException {
 +    appendProp(fileWriter, key.getKey(), value, siteConfig);
 +  }
 +  
 +  private void appendProp(Writer fileWriter, String key, String value, Map<String,String> siteConfig) throws IOException {
 +    if (!siteConfig.containsKey(key))
 +      fileWriter.append("<property><name>" + key + "</name><value>" + value + "</value></property>\n");
 +  }
 +
 +  /**
 +   * Sets a given key with a random port for the value on the site config if it doesn't already exist.
 +   */
 +  private void mergePropWithRandomPort(Map<String,String> siteConfig, String key) {
 +    if (!siteConfig.containsKey(key)) {
 +      siteConfig.put(key, "0");
 +    }
 +  }
 +  
 +  /**
 +   * 
 +   * @param dir
 +   *          An empty or nonexistant temp directoy that Accumulo and Zookeeper can store data in. Creating the directory is left to the user. Java 7, Guava,
 +   *          and Junit provide methods for creating temporary directories.
 +   * @param rootPassword
 +   *          Initial root password for instance.
 +   * @throws IOException
 +   */
 +  public MiniAccumuloCluster(File dir, String rootPassword) throws IOException {
 +    this(new MiniAccumuloConfig(dir, rootPassword));
 +  }
 +  
 +  /**
 +   * @param config
 +   *          initial configuration
 +   * @throws IOException
 +   */
 +  
 +  public MiniAccumuloCluster(MiniAccumuloConfig config) throws IOException {
 +    
 +    if (config.getDir().exists() && !config.getDir().isDirectory())
 +      throw new IllegalArgumentException("Must pass in directory, " + config.getDir() + " is a file");
 +    
 +    if (config.getDir().exists() && config.getDir().list().length != 0)
 +      throw new IllegalArgumentException("Directory " + config.getDir() + " is not empty");
 +    
 +    this.config = config;
 +    
 +    libDir = new File(config.getDir(), "lib");
 +    libExtDir = new File(libDir, "ext");
 +    confDir = new File(config.getDir(), "conf");
 +    accumuloDir = new File(config.getDir(), "accumulo");
 +    zooKeeperDir = new File(config.getDir(), "zookeeper");
 +    logDir = new File(config.getDir(), "logs");
 +    walogDir = new File(config.getDir(), "walogs");
 +    
 +    confDir.mkdirs();
 +    accumuloDir.mkdirs();
 +    zooKeeperDir.mkdirs();
 +    logDir.mkdirs();
 +    walogDir.mkdirs();
 +    libDir.mkdirs();
 +    
 +    // Avoid the classloader yelling that the general.dynamic.classpaths value is invalid because
 +    // $ACCUMULO_HOME/lib/ext isn't defined.
 +    libExtDir.mkdirs();
 +    
 +    zooKeeperPort = PortUtils.getRandomFreePort();
 +    
 +    File siteFile = new File(confDir, "accumulo-site.xml");
 +    
 +    OutputStreamWriter fileWriter = new OutputStreamWriter(new FileOutputStream(siteFile), Constants.UTF8);
 +    fileWriter.append("<configuration>\n");
 +    
 +    HashMap<String,String> siteConfig = new HashMap<String,String>(config.getSiteConfig());
 +    
 +    appendProp(fileWriter, Property.INSTANCE_DFS_URI, "file:///", siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_DFS_DIR, accumuloDir.getAbsolutePath(), siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_ZK_HOST, "localhost:" + zooKeeperPort, siteConfig);
 +    appendProp(fileWriter, Property.INSTANCE_SECRET, INSTANCE_SECRET, siteConfig);
 +    appendProp(fileWriter, Property.TSERV_PORTSEARCH, "true", siteConfig);
 +    appendProp(fileWriter, Property.LOGGER_DIR, walogDir.getAbsolutePath(), siteConfig);
 +    appendProp(fileWriter, Property.TSERV_DATACACHE_SIZE, "10M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_INDEXCACHE_SIZE, "10M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_MAXMEM, "50M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_WALOG_MAX_SIZE, "100M", siteConfig);
 +    appendProp(fileWriter, Property.TSERV_NATIVEMAP_ENABLED, "false", siteConfig);
 +    appendProp(fileWriter, Property.TRACE_TOKEN_PROPERTY_PREFIX + ".password", config.getRootPassword(), siteConfig);
 +    appendProp(fileWriter, Property.GC_CYCLE_DELAY, "4s", siteConfig);
 +    appendProp(fileWriter, Property.GC_CYCLE_START, "0s", siteConfig);
 +    mergePropWithRandomPort(siteConfig, Property.MASTER_CLIENTPORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.TRACE_PORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.TSERV_CLIENTPORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.MONITOR_PORT.getKey());
 +    mergePropWithRandomPort(siteConfig, Property.GC_PORT.getKey());
 +    
 +    // since there is a small amount of memory, check more frequently for majc... setting may not be needed in 1.5
 +    appendProp(fileWriter, Property.TSERV_MAJC_DELAY, "3", siteConfig);
 +    
 +    // ACCUMULO-1472 -- Use the classpath, not what might be installed on the system.
 +    // We have to set *something* here, otherwise the AccumuloClassLoader will default to pulling from 
 +    // environment variables (e.g. ACCUMULO_HOME, HADOOP_HOME/PREFIX) which will result in multiple copies
 +    // of artifacts on the classpath as they'll be provided by the invoking application
 +    appendProp(fileWriter, Property.GENERAL_CLASSPATHS, libDir.getAbsolutePath() + "/[^.].*.jar", siteConfig);
 +    appendProp(fileWriter, Property.GENERAL_DYNAMIC_CLASSPATHS, libExtDir.getAbsolutePath() + "/[^.].*.jar", siteConfig);
 +
 +    for (Entry<String,String> entry : siteConfig.entrySet())
 +      fileWriter.append("<property><name>" + entry.getKey() + "</name><value>" + entry.getValue() + "</value></property>\n");
 +    fileWriter.append("</configuration>\n");
 +    fileWriter.close();
 +    
 +    zooCfgFile = new File(confDir, "zoo.cfg");
 +    fileWriter = new OutputStreamWriter(new FileOutputStream(zooCfgFile), Constants.UTF8);
 +    
 +    // zookeeper uses Properties to read its config, so use that to write in order to properly escape things like Windows paths
 +    Properties zooCfg = new Properties();
 +    zooCfg.setProperty("tickTime", "1000");
 +    zooCfg.setProperty("initLimit", "10");
 +    zooCfg.setProperty("syncLimit", "5");
 +    zooCfg.setProperty("clientPort", zooKeeperPort + "");
 +    zooCfg.setProperty("maxClientCnxns", "100");
 +    zooCfg.setProperty("dataDir", zooKeeperDir.getAbsolutePath());
 +    zooCfg.store(fileWriter, null);
 +    
 +    fileWriter.close();
 +    
 +  }
 +  
 +  /**
 +   * Starts Accumulo and Zookeeper processes. Can only be called once.
 +   * 
 +   * @throws IOException
 +   * @throws InterruptedException
 +   * @throws IllegalStateException
 +   *           if already started
 +   */
 +  
 +  public void start() throws IOException, InterruptedException {
 +    if (zooKeeperProcess != null)
 +      throw new IllegalStateException("Already started");
 +    
 +    Runtime.getRuntime().addShutdownHook(new Thread() {
 +      @Override
 +      public void run() {
 +        try {
 +          MiniAccumuloCluster.this.stop();
 +        } catch (IOException e) {
 +          e.printStackTrace();
 +        } catch (InterruptedException e) {
 +          e.printStackTrace();
 +        }
 +      }
 +    });
 +    
 +    zooKeeperProcess = exec(Main.class, ZooKeeperServerMain.class.getName(), zooCfgFile.getAbsolutePath());
 +    
 +    // sleep a little bit to let zookeeper come up before calling init, seems to work better
 +    UtilWaitThread.sleep(250);
 +    
 +    Process initProcess = exec(Initialize.class, "--instance-name", INSTANCE_NAME, "--password", config.getRootPassword());
 +    int ret = initProcess.waitFor();
 +    if (ret != 0) {
 +      throw new RuntimeException("Initialize process returned " + ret + ". Check the logs in " + logDir + " for errors.");
 +    }
 +    
 +    tabletServerProcesses = new Process[config.getNumTservers()];
 +    for (int i = 0; i < config.getNumTservers(); i++) {
 +      tabletServerProcesses[i] = exec(TabletServer.class);
 +    }
 +    
 +    masterProcess = exec(Master.class);
 +    
 +    gcProcess = exec(SimpleGarbageCollector.class);
 +  }
 +  
 +  /**
 +   * @return Accumulo instance name
 +   */
 +  
 +  public String getInstanceName() {
 +    return INSTANCE_NAME;
 +  }
 +  
 +  /**
 +   * @return zookeeper connection string
 +   */
 +  
 +  public String getZooKeepers() {
 +    return "localhost:" + zooKeeperPort;
 +  }
 +  
 +  /**
 +   * Stops Accumulo and Zookeeper processes. If stop is not called, there is a shutdown hook that is setup to kill the processes. Howerver its probably best to
 +   * call stop in a finally block as soon as possible.
 +   * 
 +   * @throws IOException
 +   * @throws InterruptedException
 +   */
 +  
 +  public void stop() throws IOException, InterruptedException {
-     if (zooKeeperProcess != null)
++    if (zooKeeperProcess != null) {
 +      zooKeeperProcess.destroy();
-     if (masterProcess != null)
++      zooKeeperProcess.waitFor();
++    }
++    if (masterProcess != null) {
 +      masterProcess.destroy();
++      masterProcess.waitFor();
++    }
 +    if (tabletServerProcesses != null) {
 +      for (Process tserver : tabletServerProcesses) {
 +        tserver.destroy();
++        tserver.waitFor();
 +      }
 +    }
 +    
 +    for (LogWriter lw : logWriters)
 +      lw.flush();
 +
-     if (gcProcess != null)
++    if (gcProcess != null) {
 +      gcProcess.destroy();
++      gcProcess.waitFor();
++    }
 +  }
 +}


[03/10] git commit: ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()

Posted by el...@apache.org.
ACCUMULO-2512 Wait for each MAC process to exit before returning from MAC.stop()


Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo
Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/0753a754
Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/0753a754
Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/0753a754

Branch: refs/heads/1.6.0-SNAPSHOT
Commit: 0753a754b0a1751ab7f2bb4ddd922aa9dd492252
Parents: bbd6aeb
Author: Josh Elser <el...@apache.org>
Authored: Thu Mar 20 16:35:04 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Mar 20 16:35:04 2014 -0400

----------------------------------------------------------------------
 .../accumulo/minicluster/MiniAccumuloCluster.java  | 17 +++++++++++++----
 1 file changed, 13 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/0753a754/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
----------------------------------------------------------------------
diff --git a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
index cbed375..848e14f 100644
--- a/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
+++ b/src/minicluster/src/main/java/org/apache/accumulo/minicluster/MiniAccumuloCluster.java
@@ -362,22 +362,31 @@ public class MiniAccumuloCluster {
    */
   
   public void stop() throws IOException, InterruptedException {
-    if (zooKeeperProcess != null)
+    if (zooKeeperProcess != null) {
       zooKeeperProcess.destroy();
-    if (loggerProcess != null)
+      zooKeeperProcess.waitFor();
+    }
+    if (loggerProcess != null) {
       loggerProcess.destroy();
-    if (masterProcess != null)
+      loggerProcess.waitFor();
+    }
+    if (masterProcess != null) {
       masterProcess.destroy();
+      masterProcess.waitFor();
+    }
     if (tabletServerProcesses != null) {
       for (Process tserver : tabletServerProcesses) {
         tserver.destroy();
+        tserver.waitFor();
       }
     }
     
     for (LogWriter lw : logWriters)
       lw.flush();
 
-    if (gcProcess != null)
+    if (gcProcess != null) {
       gcProcess.destroy();
+      gcProcess.waitFor();
+    }
   }
 }