001/*
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.zookeeper;
020
021import java.io.BufferedReader;
022import java.io.BufferedWriter;
023import java.io.IOException;
024import java.io.InputStreamReader;
025import java.io.OutputStreamWriter;
026import java.io.PrintWriter;
027import java.net.InetSocketAddress;
028import java.net.Socket;
029import java.nio.charset.StandardCharsets;
030import java.util.ArrayList;
031import java.util.Arrays;
032import java.util.Collections;
033import java.util.Deque;
034import java.util.HashMap;
035import java.util.LinkedList;
036import java.util.List;
037import java.util.Map;
038import javax.security.auth.login.AppConfigurationEntry;
039import javax.security.auth.login.AppConfigurationEntry.LoginModuleControlFlag;
040import org.apache.commons.lang3.StringUtils;
041import org.apache.hadoop.conf.Configuration;
042import org.apache.hadoop.hbase.AuthUtil;
043import org.apache.hadoop.hbase.HBaseConfiguration;
044import org.apache.hadoop.hbase.HConstants;
045import org.apache.hadoop.hbase.exceptions.DeserializationException;
046import org.apache.hadoop.hbase.security.Superusers;
047import org.apache.hadoop.hbase.util.Bytes;
048import org.apache.hadoop.hbase.util.Threads;
049import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp.CreateAndFailSilent;
050import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp.DeleteNodeFailSilent;
051import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp.SetData;
052import org.apache.hadoop.security.SecurityUtil;
053import org.apache.hadoop.security.UserGroupInformation;
054import org.apache.hadoop.security.authentication.util.KerberosUtil;
055import org.apache.yetus.audience.InterfaceAudience;
056import org.apache.zookeeper.AsyncCallback;
057import org.apache.zookeeper.CreateMode;
058import org.apache.zookeeper.KeeperException;
059import org.apache.zookeeper.KeeperException.NoNodeException;
060import org.apache.zookeeper.Op;
061import org.apache.zookeeper.Watcher;
062import org.apache.zookeeper.ZooDefs.Ids;
063import org.apache.zookeeper.ZooDefs.Perms;
064import org.apache.zookeeper.ZooKeeper;
065import org.apache.zookeeper.client.ZooKeeperSaslClient;
066import org.apache.zookeeper.data.ACL;
067import org.apache.zookeeper.data.Id;
068import org.apache.zookeeper.data.Stat;
069import org.apache.zookeeper.proto.CreateRequest;
070import org.apache.zookeeper.proto.DeleteRequest;
071import org.apache.zookeeper.proto.SetDataRequest;
072import org.apache.zookeeper.server.ZooKeeperSaslServer;
073import org.slf4j.Logger;
074import org.slf4j.LoggerFactory;
075
076import org.apache.hbase.thirdparty.com.google.protobuf.InvalidProtocolBufferException;
077
078import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
079import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
080
081/**
082 * Internal HBase utility class for ZooKeeper.
083 *
084 * <p>Contains only static methods and constants.
085 *
086 * <p>Methods all throw {@link KeeperException} if there is an unexpected
087 * zookeeper exception, so callers of these methods must handle appropriately.
088 * If ZK is required for the operation, the server will need to be aborted.
089 */
090@InterfaceAudience.Private
091public final class ZKUtil {
092  private static final Logger LOG = LoggerFactory.getLogger(ZKUtil.class);
093
094  private static int zkDumpConnectionTimeOut;
095
096  private ZKUtil() {
097  }
098
099  /**
100   * Creates a new connection to ZooKeeper, pulling settings and ensemble config
101   * from the specified configuration object using methods from {@link ZKConfig}.
102   *
103   * Sets the connection status monitoring watcher to the specified watcher.
104   *
105   * @param conf configuration to pull ensemble and other settings from
106   * @param watcher watcher to monitor connection changes
107   * @return connection to zookeeper
108   * @throws IOException if unable to connect to zk or config problem
109   */
110  public static RecoverableZooKeeper connect(Configuration conf, Watcher watcher)
111    throws IOException {
112    String ensemble = ZKConfig.getZKQuorumServersString(conf);
113    return connect(conf, ensemble, watcher);
114  }
115
116  public static RecoverableZooKeeper connect(Configuration conf, String ensemble,
117      Watcher watcher)
118    throws IOException {
119    return connect(conf, ensemble, watcher, null);
120  }
121
122  public static RecoverableZooKeeper connect(Configuration conf, String ensemble,
123      Watcher watcher, final String identifier)
124    throws IOException {
125    if(ensemble == null) {
126      throw new IOException("Unable to determine ZooKeeper ensemble");
127    }
128    int timeout = conf.getInt(HConstants.ZK_SESSION_TIMEOUT,
129        HConstants.DEFAULT_ZK_SESSION_TIMEOUT);
130    if (LOG.isTraceEnabled()) {
131      LOG.trace("{} opening connection to ZooKeeper ensemble={}", identifier, ensemble);
132    }
133    int retry = conf.getInt("zookeeper.recovery.retry", 3);
134    int retryIntervalMillis =
135      conf.getInt("zookeeper.recovery.retry.intervalmill", 1000);
136    int maxSleepTime = conf.getInt("zookeeper.recovery.retry.maxsleeptime", 60000);
137    zkDumpConnectionTimeOut = conf.getInt("zookeeper.dump.connection.timeout",
138        1000);
139    return new RecoverableZooKeeper(ensemble, timeout, watcher,
140        retry, retryIntervalMillis, maxSleepTime, identifier);
141  }
142
143  /**
144   * Log in the current zookeeper server process using the given configuration
145   * keys for the credential file and login principal.
146   *
147   * <p><strong>This is only applicable when running on secure hbase</strong>
148   * On regular HBase (without security features), this will safely be ignored.
149   * </p>
150   *
151   * @param conf The configuration data to use
152   * @param keytabFileKey Property key used to configure the path to the credential file
153   * @param userNameKey Property key used to configure the login principal
154   * @param hostname Current hostname to use in any credentials
155   * @throws IOException underlying exception from SecurityUtil.login() call
156   */
157  public static void loginServer(Configuration conf, String keytabFileKey,
158      String userNameKey, String hostname) throws IOException {
159    login(conf, keytabFileKey, userNameKey, hostname,
160          ZooKeeperSaslServer.LOGIN_CONTEXT_NAME_KEY,
161          JaasConfiguration.SERVER_KEYTAB_KERBEROS_CONFIG_NAME);
162  }
163
164  /**
165   * Log in the current zookeeper client using the given configuration
166   * keys for the credential file and login principal.
167   *
168   * <p><strong>This is only applicable when running on secure hbase</strong>
169   * On regular HBase (without security features), this will safely be ignored.
170   * </p>
171   *
172   * @param conf The configuration data to use
173   * @param keytabFileKey Property key used to configure the path to the credential file
174   * @param userNameKey Property key used to configure the login principal
175   * @param hostname Current hostname to use in any credentials
176   * @throws IOException underlying exception from SecurityUtil.login() call
177   */
178  public static void loginClient(Configuration conf, String keytabFileKey,
179      String userNameKey, String hostname) throws IOException {
180    login(conf, keytabFileKey, userNameKey, hostname,
181          ZooKeeperSaslClient.LOGIN_CONTEXT_NAME_KEY,
182          JaasConfiguration.CLIENT_KEYTAB_KERBEROS_CONFIG_NAME);
183  }
184
185  /**
186   * Log in the current process using the given configuration keys for the
187   * credential file and login principal.
188   *
189   * <p><strong>This is only applicable when running on secure hbase</strong>
190   * On regular HBase (without security features), this will safely be ignored.
191   * </p>
192   *
193   * @param conf The configuration data to use
194   * @param keytabFileKey Property key used to configure the path to the credential file
195   * @param userNameKey Property key used to configure the login principal
196   * @param hostname Current hostname to use in any credentials
197   * @param loginContextProperty property name to expose the entry name
198   * @param loginContextName jaas entry name
199   * @throws IOException underlying exception from SecurityUtil.login() call
200   */
201  private static void login(Configuration conf, String keytabFileKey,
202      String userNameKey, String hostname,
203      String loginContextProperty, String loginContextName)
204      throws IOException {
205    if (!isSecureZooKeeper(conf)) {
206      return;
207    }
208
209    // User has specified a jaas.conf, keep this one as the good one.
210    // HBASE_OPTS="-Djava.security.auth.login.config=jaas.conf"
211    if (System.getProperty("java.security.auth.login.config") != null) {
212      return;
213    }
214
215    // No keytab specified, no auth
216    String keytabFilename = conf.get(keytabFileKey);
217    if (keytabFilename == null) {
218      LOG.warn("no keytab specified for: {}", keytabFileKey);
219      return;
220    }
221
222    String principalConfig = conf.get(userNameKey, System.getProperty("user.name"));
223    String principalName = SecurityUtil.getServerPrincipal(principalConfig, hostname);
224
225    // Initialize the "jaas.conf" for keyTab/principal,
226    // If keyTab is not specified use the Ticket Cache.
227    // and set the zookeeper login context name.
228    JaasConfiguration jaasConf = new JaasConfiguration(loginContextName,
229        principalName, keytabFilename);
230    javax.security.auth.login.Configuration.setConfiguration(jaasConf);
231    System.setProperty(loginContextProperty, loginContextName);
232  }
233
234  /**
235   * A JAAS configuration that defines the login modules that we want to use for login.
236   */
237  private static class JaasConfiguration extends javax.security.auth.login.Configuration {
238    private static final String SERVER_KEYTAB_KERBEROS_CONFIG_NAME =
239      "zookeeper-server-keytab-kerberos";
240    private static final String CLIENT_KEYTAB_KERBEROS_CONFIG_NAME =
241      "zookeeper-client-keytab-kerberos";
242
243    private static final Map<String, String> BASIC_JAAS_OPTIONS = new HashMap<>();
244    static {
245      String jaasEnvVar = System.getenv("HBASE_JAAS_DEBUG");
246      if ("true".equalsIgnoreCase(jaasEnvVar)) {
247        BASIC_JAAS_OPTIONS.put("debug", "true");
248      }
249    }
250
251    private static final Map<String,String> KEYTAB_KERBEROS_OPTIONS = new HashMap<>();
252    static {
253      KEYTAB_KERBEROS_OPTIONS.put("doNotPrompt", "true");
254      KEYTAB_KERBEROS_OPTIONS.put("storeKey", "true");
255      KEYTAB_KERBEROS_OPTIONS.put("refreshKrb5Config", "true");
256      KEYTAB_KERBEROS_OPTIONS.putAll(BASIC_JAAS_OPTIONS);
257    }
258
259    private static final AppConfigurationEntry KEYTAB_KERBEROS_LOGIN =
260      new AppConfigurationEntry(KerberosUtil.getKrb5LoginModuleName(),
261                                LoginModuleControlFlag.REQUIRED,
262                                KEYTAB_KERBEROS_OPTIONS);
263
264    private static final AppConfigurationEntry[] KEYTAB_KERBEROS_CONF =
265      new AppConfigurationEntry[]{KEYTAB_KERBEROS_LOGIN};
266
267    private javax.security.auth.login.Configuration baseConfig;
268    private final String loginContextName;
269    private final boolean useTicketCache;
270    private final String keytabFile;
271    private final String principal;
272
273    public JaasConfiguration(String loginContextName, String principal, String keytabFile) {
274      this(loginContextName, principal, keytabFile, keytabFile == null || keytabFile.length() == 0);
275    }
276
277    private JaasConfiguration(String loginContextName, String principal,
278                             String keytabFile, boolean useTicketCache) {
279      try {
280        this.baseConfig = javax.security.auth.login.Configuration.getConfiguration();
281      } catch (SecurityException e) {
282        this.baseConfig = null;
283      }
284      this.loginContextName = loginContextName;
285      this.useTicketCache = useTicketCache;
286      this.keytabFile = keytabFile;
287      this.principal = principal;
288      LOG.info("JaasConfiguration loginContextName={} principal={} useTicketCache={} keytabFile={}",
289        loginContextName, principal, useTicketCache, keytabFile);
290    }
291
292    @Override
293    public AppConfigurationEntry[] getAppConfigurationEntry(String appName) {
294      if (loginContextName.equals(appName)) {
295        if (!useTicketCache) {
296          KEYTAB_KERBEROS_OPTIONS.put("keyTab", keytabFile);
297          KEYTAB_KERBEROS_OPTIONS.put("useKeyTab", "true");
298        }
299        KEYTAB_KERBEROS_OPTIONS.put("principal", principal);
300        KEYTAB_KERBEROS_OPTIONS.put("useTicketCache", useTicketCache ? "true" : "false");
301        return KEYTAB_KERBEROS_CONF;
302      }
303
304      if (baseConfig != null) {
305        return baseConfig.getAppConfigurationEntry(appName);
306      }
307
308      return(null);
309    }
310  }
311
312  //
313  // Helper methods
314  //
315  /**
316   * Returns the full path of the immediate parent of the specified node.
317   * @param node path to get parent of
318   * @return parent of path, null if passed the root node or an invalid node
319   */
320  public static String getParent(String node) {
321    int idx = node.lastIndexOf(ZNodePaths.ZNODE_PATH_SEPARATOR);
322    return idx <= 0 ? null : node.substring(0, idx);
323  }
324
325  /**
326   * Get the name of the current node from the specified fully-qualified path.
327   * @param path fully-qualified path
328   * @return name of the current node
329   */
330  public static String getNodeName(String path) {
331    return path.substring(path.lastIndexOf("/")+1);
332  }
333
334  //
335  // Existence checks and watches
336  //
337
338  /**
339   * Watch the specified znode for delete/create/change events.  The watcher is
340   * set whether or not the node exists.  If the node already exists, the method
341   * returns true.  If the node does not exist, the method returns false.
342   *
343   * @param zkw zk reference
344   * @param znode path of node to watch
345   * @return true if znode exists, false if does not exist or error
346   * @throws KeeperException if unexpected zookeeper exception
347   */
348  public static boolean watchAndCheckExists(ZKWatcher zkw, String znode)
349    throws KeeperException {
350    try {
351      Stat s = zkw.getRecoverableZooKeeper().exists(znode, zkw);
352      boolean exists = s != null;
353      if (exists) {
354        LOG.debug(zkw.prefix("Set watcher on existing znode=" + znode));
355      } else {
356        LOG.debug(zkw.prefix("Set watcher on znode that does not yet exist, " + znode));
357      }
358      return exists;
359    } catch (KeeperException e) {
360      LOG.warn(zkw.prefix("Unable to set watcher on znode " + znode), e);
361      zkw.keeperException(e);
362      return false;
363    } catch (InterruptedException e) {
364      LOG.warn(zkw.prefix("Unable to set watcher on znode " + znode), e);
365      zkw.interruptedException(e);
366      return false;
367    }
368  }
369
370  /**
371   * Watch the specified znode, but only if exists. Useful when watching
372   * for deletions. Uses .getData() (and handles NoNodeException) instead
373   * of .exists() to accomplish this, as .getData() will only set a watch if
374   * the znode exists.
375   * @param zkw zk reference
376   * @param znode path of node to watch
377   * @return true if the watch is set, false if node does not exists
378   * @throws KeeperException if unexpected zookeeper exception
379   */
380  public static boolean setWatchIfNodeExists(ZKWatcher zkw, String znode)
381      throws KeeperException {
382    try {
383      zkw.getRecoverableZooKeeper().getData(znode, true, null);
384      return true;
385    } catch (NoNodeException e) {
386      return false;
387    } catch (InterruptedException e) {
388      LOG.warn(zkw.prefix("Unable to set watcher on znode " + znode), e);
389      zkw.interruptedException(e);
390      return false;
391    }
392  }
393
394  /**
395   * Check if the specified node exists.  Sets no watches.
396   *
397   * @param zkw zk reference
398   * @param znode path of node to watch
399   * @return version of the node if it exists, -1 if does not exist
400   * @throws KeeperException if unexpected zookeeper exception
401   */
402  public static int checkExists(ZKWatcher zkw, String znode)
403    throws KeeperException {
404    try {
405      Stat s = zkw.getRecoverableZooKeeper().exists(znode, null);
406      return s != null ? s.getVersion() : -1;
407    } catch (KeeperException e) {
408      LOG.warn(zkw.prefix("Unable to set watcher on znode (" + znode + ")"), e);
409      zkw.keeperException(e);
410      return -1;
411    } catch (InterruptedException e) {
412      LOG.warn(zkw.prefix("Unable to set watcher on znode (" + znode + ")"), e);
413      zkw.interruptedException(e);
414      return -1;
415    }
416  }
417
418  //
419  // Znode listings
420  //
421
422  /**
423   * Lists the children znodes of the specified znode.  Also sets a watch on
424   * the specified znode which will capture a NodeDeleted event on the specified
425   * znode as well as NodeChildrenChanged if any children of the specified znode
426   * are created or deleted.
427   *
428   * Returns null if the specified node does not exist.  Otherwise returns a
429   * list of children of the specified node.  If the node exists but it has no
430   * children, an empty list will be returned.
431   *
432   * @param zkw zk reference
433   * @param znode path of node to list and watch children of
434   * @return list of children of the specified node, an empty list if the node
435   *          exists but has no children, and null if the node does not exist
436   * @throws KeeperException if unexpected zookeeper exception
437   */
438  public static List<String> listChildrenAndWatchForNewChildren(
439          ZKWatcher zkw, String znode)
440    throws KeeperException {
441    try {
442      return zkw.getRecoverableZooKeeper().getChildren(znode, zkw);
443    } catch(KeeperException.NoNodeException ke) {
444      LOG.debug(zkw.prefix("Unable to list children of znode " + znode + " " +
445          "because node does not exist (not an error)"));
446      return null;
447    } catch (KeeperException e) {
448      LOG.warn(zkw.prefix("Unable to list children of znode " + znode + " "), e);
449      zkw.keeperException(e);
450      return null;
451    } catch (InterruptedException e) {
452      LOG.warn(zkw.prefix("Unable to list children of znode " + znode + " "), e);
453      zkw.interruptedException(e);
454      return null;
455    }
456  }
457
458  /**
459   * List all the children of the specified znode, setting a watch for children
460   * changes and also setting a watch on every individual child in order to get
461   * the NodeCreated and NodeDeleted events.
462   * @param zkw zookeeper reference
463   * @param znode node to get children of and watch
464   * @return list of znode names, null if the node doesn't exist
465   * @throws KeeperException if a ZooKeeper operation fails
466   */
467  public static List<String> listChildrenAndWatchThem(ZKWatcher zkw,
468      String znode) throws KeeperException {
469    List<String> children = listChildrenAndWatchForNewChildren(zkw, znode);
470    if (children == null) {
471      return null;
472    }
473    for (String child : children) {
474      watchAndCheckExists(zkw, ZNodePaths.joinZNode(znode, child));
475    }
476    return children;
477  }
478
479  /**
480   * Lists the children of the specified znode without setting any watches.
481   *
482   * Sets no watches at all, this method is best effort.
483   *
484   * Returns an empty list if the node has no children.  Returns null if the
485   * parent node itself does not exist.
486   *
487   * @param zkw zookeeper reference
488   * @param znode node to get children
489   * @return list of data of children of specified znode, empty if no children,
490   *         null if parent does not exist
491   * @throws KeeperException if unexpected zookeeper exception
492   */
493  public static List<String> listChildrenNoWatch(ZKWatcher zkw, String znode)
494    throws KeeperException {
495    List<String> children = null;
496    try {
497      // List the children without watching
498      children = zkw.getRecoverableZooKeeper().getChildren(znode, null);
499    } catch(KeeperException.NoNodeException nne) {
500      return null;
501    } catch(InterruptedException ie) {
502      zkw.interruptedException(ie);
503    }
504    return children;
505  }
506
507  /**
508   * Simple class to hold a node path and node data.
509   * @deprecated Unused
510   */
511  @Deprecated
512  public static class NodeAndData {
513    private String node;
514    private byte [] data;
515    public NodeAndData(String node, byte [] data) {
516      this.node = node;
517      this.data = data;
518    }
519    public String getNode() {
520      return node;
521    }
522    public byte [] getData() {
523      return data;
524    }
525    @Override
526    public String toString() {
527      return node;
528    }
529    public boolean isEmpty() {
530      return (data == null || data.length == 0);
531    }
532  }
533
534  /**
535   * Checks if the specified znode has any children.  Sets no watches.
536   *
537   * Returns true if the node exists and has children.  Returns false if the
538   * node does not exist or if the node does not have any children.
539   *
540   * Used during master initialization to determine if the master is a
541   * failed-over-to master or the first master during initial cluster startup.
542   * If the directory for regionserver ephemeral nodes is empty then this is
543   * a cluster startup, if not then it is not cluster startup.
544   *
545   * @param zkw zk reference
546   * @param znode path of node to check for children of
547   * @return true if node has children, false if not or node does not exist
548   * @throws KeeperException if unexpected zookeeper exception
549   */
550  public static boolean nodeHasChildren(ZKWatcher zkw, String znode)
551    throws KeeperException {
552    try {
553      return !zkw.getRecoverableZooKeeper().getChildren(znode, null).isEmpty();
554    } catch(KeeperException.NoNodeException ke) {
555      LOG.debug(zkw.prefix("Unable to list children of znode " + znode +
556              " because node does not exist (not an error)"));
557      return false;
558    } catch (KeeperException e) {
559      LOG.warn(zkw.prefix("Unable to list children of znode " + znode), e);
560      zkw.keeperException(e);
561      return false;
562    } catch (InterruptedException e) {
563      LOG.warn(zkw.prefix("Unable to list children of znode " + znode), e);
564      zkw.interruptedException(e);
565      return false;
566    }
567  }
568
569  /**
570   * Get the number of children of the specified node.
571   *
572   * If the node does not exist or has no children, returns 0.
573   *
574   * Sets no watches at all.
575   *
576   * @param zkw zk reference
577   * @param znode path of node to count children of
578   * @return number of children of specified node, 0 if none or parent does not
579   *         exist
580   * @throws KeeperException if unexpected zookeeper exception
581   */
582  public static int getNumberOfChildren(ZKWatcher zkw, String znode)
583    throws KeeperException {
584    try {
585      Stat stat = zkw.getRecoverableZooKeeper().exists(znode, null);
586      return stat == null ? 0 : stat.getNumChildren();
587    } catch(KeeperException e) {
588      LOG.warn(zkw.prefix("Unable to get children of node " + znode));
589      zkw.keeperException(e);
590    } catch(InterruptedException e) {
591      zkw.interruptedException(e);
592    }
593    return 0;
594  }
595
596  //
597  // Data retrieval
598  //
599
600  /**
601   * Get znode data. Does not set a watcher.
602   *
603   * @return ZNode data, null if the node does not exist or if there is an error.
604   */
605  public static byte [] getData(ZKWatcher zkw, String znode)
606      throws KeeperException, InterruptedException {
607    try {
608      byte [] data = zkw.getRecoverableZooKeeper().getData(znode, null, null);
609      logRetrievedMsg(zkw, znode, data, false);
610      return data;
611    } catch (KeeperException.NoNodeException e) {
612      LOG.debug(zkw.prefix("Unable to get data of znode " + znode + " " +
613          "because node does not exist (not an error)"));
614      return null;
615    } catch (KeeperException e) {
616      LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
617      zkw.keeperException(e);
618      return null;
619    }
620  }
621
622  /**
623   * Get the data at the specified znode and set a watch.
624   *
625   * Returns the data and sets a watch if the node exists.  Returns null and no
626   * watch is set if the node does not exist or there is an exception.
627   *
628   * @param zkw zk reference
629   * @param znode path of node
630   * @return data of the specified znode, or null
631   * @throws KeeperException if unexpected zookeeper exception
632   */
633  public static byte [] getDataAndWatch(ZKWatcher zkw, String znode)
634    throws KeeperException {
635    return getDataInternal(zkw, znode, null, true);
636  }
637
638  /**
639   * Get the data at the specified znode and set a watch.
640   *
641   * Returns the data and sets a watch if the node exists.  Returns null and no
642   * watch is set if the node does not exist or there is an exception.
643   *
644   * @param zkw zk reference
645   * @param znode path of node
646   * @param stat object to populate the version of the znode
647   * @return data of the specified znode, or null
648   * @throws KeeperException if unexpected zookeeper exception
649   */
650  public static byte[] getDataAndWatch(ZKWatcher zkw, String znode,
651                                       Stat stat) throws KeeperException {
652    return getDataInternal(zkw, znode, stat, true);
653  }
654
655  private static byte[] getDataInternal(ZKWatcher zkw, String znode, Stat stat,
656                                        boolean watcherSet)
657      throws KeeperException {
658    try {
659      byte [] data = zkw.getRecoverableZooKeeper().getData(znode, zkw, stat);
660      logRetrievedMsg(zkw, znode, data, watcherSet);
661      return data;
662    } catch (KeeperException.NoNodeException e) {
663      // This log can get pretty annoying when we cycle on 100ms waits.
664      // Enable trace if you really want to see it.
665      LOG.trace(zkw.prefix("Unable to get data of znode " + znode + " " +
666        "because node does not exist (not an error)"));
667      return null;
668    } catch (KeeperException e) {
669      LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
670      zkw.keeperException(e);
671      return null;
672    } catch (InterruptedException e) {
673      LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
674      zkw.interruptedException(e);
675      return null;
676    }
677  }
678
679  /**
680   * Get the data at the specified znode without setting a watch.
681   *
682   * Returns the data if the node exists.  Returns null if the node does not
683   * exist.
684   *
685   * Sets the stats of the node in the passed Stat object.  Pass a null stat if
686   * not interested.
687   *
688   * @param zkw zk reference
689   * @param znode path of node
690   * @param stat node status to get if node exists
691   * @return data of the specified znode, or null if node does not exist
692   * @throws KeeperException if unexpected zookeeper exception
693   */
694  public static byte [] getDataNoWatch(ZKWatcher zkw, String znode,
695                                       Stat stat)
696    throws KeeperException {
697    try {
698      byte [] data = zkw.getRecoverableZooKeeper().getData(znode, null, stat);
699      logRetrievedMsg(zkw, znode, data, false);
700      return data;
701    } catch (KeeperException.NoNodeException e) {
702      LOG.debug(zkw.prefix("Unable to get data of znode " + znode + " " +
703          "because node does not exist (not necessarily an error)"));
704      return null;
705    } catch (KeeperException e) {
706      LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
707      zkw.keeperException(e);
708      return null;
709    } catch (InterruptedException e) {
710      LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
711      zkw.interruptedException(e);
712      return null;
713    }
714  }
715
716  /**
717   * Returns the date of child znodes of the specified znode.  Also sets a watch on
718   * the specified znode which will capture a NodeDeleted event on the specified
719   * znode as well as NodeChildrenChanged if any children of the specified znode
720   * are created or deleted.
721   *
722   * Returns null if the specified node does not exist.  Otherwise returns a
723   * list of children of the specified node.  If the node exists but it has no
724   * children, an empty list will be returned.
725   *
726   * @param zkw zk reference
727   * @param baseNode path of node to list and watch children of
728   * @return list of data of children of the specified node, an empty list if the node
729   *          exists but has no children, and null if the node does not exist
730   * @throws KeeperException if unexpected zookeeper exception
731   * @deprecated Unused
732   */
733  @Deprecated
734  public static List<NodeAndData> getChildDataAndWatchForNewChildren(
735          ZKWatcher zkw, String baseNode) throws KeeperException {
736    List<String> nodes =
737      ZKUtil.listChildrenAndWatchForNewChildren(zkw, baseNode);
738    if (nodes != null) {
739      List<NodeAndData> newNodes = new ArrayList<>();
740      for (String node : nodes) {
741        String nodePath = ZNodePaths.joinZNode(baseNode, node);
742        byte[] data = ZKUtil.getDataAndWatch(zkw, nodePath);
743        newNodes.add(new NodeAndData(nodePath, data));
744      }
745      return newNodes;
746    }
747    return null;
748  }
749
750  /**
751   * Update the data of an existing node with the expected version to have the
752   * specified data.
753   *
754   * Throws an exception if there is a version mismatch or some other problem.
755   *
756   * Sets no watches under any conditions.
757   *
758   * @param zkw zk reference
759   * @param znode the path to the ZNode
760   * @param data the data to store in ZooKeeper
761   * @param expectedVersion the expected version
762   * @throws KeeperException if unexpected zookeeper exception
763   * @throws KeeperException.BadVersionException if version mismatch
764   * @deprecated Unused
765   */
766  @Deprecated
767  public static void updateExistingNodeData(ZKWatcher zkw, String znode, byte[] data,
768      int expectedVersion) throws KeeperException {
769    try {
770      zkw.getRecoverableZooKeeper().setData(znode, data, expectedVersion);
771    } catch(InterruptedException ie) {
772      zkw.interruptedException(ie);
773    }
774  }
775
776  //
777  // Data setting
778  //
779
780  /**
781   * Sets the data of the existing znode to be the specified data.  Ensures that
782   * the current data has the specified expected version.
783   *
784   * <p>If the node does not exist, a {@link NoNodeException} will be thrown.
785   *
786   * <p>If their is a version mismatch, method returns null.
787   *
788   * <p>No watches are set but setting data will trigger other watchers of this
789   * node.
790   *
791   * <p>If there is another problem, a KeeperException will be thrown.
792   *
793   * @param zkw zk reference
794   * @param znode path of node
795   * @param data data to set for node
796   * @param expectedVersion version expected when setting data
797   * @return true if data set, false if version mismatch
798   * @throws KeeperException if unexpected zookeeper exception
799   */
800  public static boolean setData(ZKWatcher zkw, String znode,
801                                byte [] data, int expectedVersion)
802    throws KeeperException, KeeperException.NoNodeException {
803    try {
804      return zkw.getRecoverableZooKeeper().setData(znode, data, expectedVersion) != null;
805    } catch (InterruptedException e) {
806      zkw.interruptedException(e);
807      return false;
808    }
809  }
810
811  /**
812   * Set data into node creating node if it doesn't yet exist.
813   * Does not set watch.
814   *
815   * @param zkw zk reference
816   * @param znode path of node
817   * @param data data to set for node
818   * @throws KeeperException if a ZooKeeper operation fails
819   */
820  public static void createSetData(final ZKWatcher zkw, final String znode, final byte [] data)
821          throws KeeperException {
822    if (checkExists(zkw, znode) == -1) {
823      ZKUtil.createWithParents(zkw, znode, data);
824    } else {
825      ZKUtil.setData(zkw, znode, data);
826    }
827  }
828
829  /**
830   * Sets the data of the existing znode to be the specified data.  The node
831   * must exist but no checks are done on the existing data or version.
832   *
833   * <p>If the node does not exist, a {@link NoNodeException} will be thrown.
834   *
835   * <p>No watches are set but setting data will trigger other watchers of this
836   * node.
837   *
838   * <p>If there is another problem, a KeeperException will be thrown.
839   *
840   * @param zkw zk reference
841   * @param znode path of node
842   * @param data data to set for node
843   * @throws KeeperException if unexpected zookeeper exception
844   */
845  public static void setData(ZKWatcher zkw, String znode, byte [] data)
846    throws KeeperException, KeeperException.NoNodeException {
847    setData(zkw, (SetData)ZKUtilOp.setData(znode, data));
848  }
849
850  private static void setData(ZKWatcher zkw, SetData setData)
851    throws KeeperException, KeeperException.NoNodeException {
852    SetDataRequest sd = (SetDataRequest)toZooKeeperOp(zkw, setData).toRequestRecord();
853    setData(zkw, sd.getPath(), sd.getData(), sd.getVersion());
854  }
855
856  /**
857   * Returns whether or not secure authentication is enabled
858   * (whether <code>hbase.security.authentication</code> is set to
859   * <code>kerberos</code>.
860   */
861  public static boolean isSecureZooKeeper(Configuration conf) {
862    // Detection for embedded HBase client with jaas configuration
863    // defined for third party programs.
864    try {
865      javax.security.auth.login.Configuration testConfig =
866          javax.security.auth.login.Configuration.getConfiguration();
867      if (testConfig.getAppConfigurationEntry("Client") == null
868          && testConfig.getAppConfigurationEntry(
869            JaasConfiguration.CLIENT_KEYTAB_KERBEROS_CONFIG_NAME) == null
870          && testConfig.getAppConfigurationEntry(
871              JaasConfiguration.SERVER_KEYTAB_KERBEROS_CONFIG_NAME) == null
872          && conf.get(HConstants.ZK_CLIENT_KERBEROS_PRINCIPAL) == null
873          && conf.get(HConstants.ZK_SERVER_KERBEROS_PRINCIPAL) == null) {
874
875        return false;
876      }
877    } catch(Exception e) {
878      // No Jaas configuration defined.
879      return false;
880    }
881
882    // Master & RSs uses hbase.zookeeper.client.*
883    return "kerberos".equalsIgnoreCase(conf.get("hbase.security.authentication"));
884  }
885
886  private static ArrayList<ACL> createACL(ZKWatcher zkw, String node) {
887    return createACL(zkw, node, isSecureZooKeeper(zkw.getConfiguration()));
888  }
889
890  public static ArrayList<ACL> createACL(ZKWatcher zkw, String node,
891                                         boolean isSecureZooKeeper) {
892    if (!node.startsWith(zkw.getZNodePaths().baseZNode)) {
893      return Ids.OPEN_ACL_UNSAFE;
894    }
895    if (isSecureZooKeeper) {
896      ArrayList<ACL> acls = new ArrayList<>();
897      // add permission to hbase supper user
898      String[] superUsers = zkw.getConfiguration().getStrings(Superusers.SUPERUSER_CONF_KEY);
899      String hbaseUser = null;
900      try {
901        hbaseUser = UserGroupInformation.getCurrentUser().getShortUserName();
902      } catch (IOException e) {
903        LOG.debug("Could not acquire current User.", e);
904      }
905      if (superUsers != null) {
906        List<String> groups = new ArrayList<>();
907        for (String user : superUsers) {
908          if (AuthUtil.isGroupPrincipal(user)) {
909            // TODO: Set node ACL for groups when ZK supports this feature
910            groups.add(user);
911          } else {
912            if(!user.equals(hbaseUser)) {
913              acls.add(new ACL(Perms.ALL, new Id("sasl", user)));
914            }
915          }
916        }
917        if (!groups.isEmpty()) {
918          LOG.warn("Znode ACL setting for group {} is skipped, ZooKeeper doesn't support this " +
919            "feature presently.", groups);
920        }
921      }
922      // Certain znodes are accessed directly by the client,
923      // so they must be readable by non-authenticated clients
924      if (zkw.getZNodePaths().isClientReadable(node)) {
925        acls.addAll(Ids.CREATOR_ALL_ACL);
926        acls.addAll(Ids.READ_ACL_UNSAFE);
927      } else {
928        acls.addAll(Ids.CREATOR_ALL_ACL);
929      }
930      return acls;
931    } else {
932      return Ids.OPEN_ACL_UNSAFE;
933    }
934  }
935
936  //
937  // Node creation
938  //
939
940  /**
941   *
942   * Set the specified znode to be an ephemeral node carrying the specified
943   * data.
944   *
945   * If the node is created successfully, a watcher is also set on the node.
946   *
947   * If the node is not created successfully because it already exists, this
948   * method will also set a watcher on the node.
949   *
950   * If there is another problem, a KeeperException will be thrown.
951   *
952   * @param zkw zk reference
953   * @param znode path of node
954   * @param data data of node
955   * @return true if node created, false if not, watch set in both cases
956   * @throws KeeperException if unexpected zookeeper exception
957   */
958  public static boolean createEphemeralNodeAndWatch(ZKWatcher zkw,
959      String znode, byte [] data)
960    throws KeeperException {
961    boolean ret = true;
962    try {
963      zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
964          CreateMode.EPHEMERAL);
965    } catch (KeeperException.NodeExistsException nee) {
966      ret = false;
967    } catch (InterruptedException e) {
968      LOG.info("Interrupted", e);
969      Thread.currentThread().interrupt();
970    }
971    if(!watchAndCheckExists(zkw, znode)) {
972      // It did exist but now it doesn't, try again
973      return createEphemeralNodeAndWatch(zkw, znode, data);
974    }
975    return ret;
976  }
977
978  /**
979   * Creates the specified znode to be a persistent node carrying the specified
980   * data.
981   *
982   * Returns true if the node was successfully created, false if the node
983   * already existed.
984   *
985   * If the node is created successfully, a watcher is also set on the node.
986   *
987   * If the node is not created successfully because it already exists, this
988   * method will also set a watcher on the node but return false.
989   *
990   * If there is another problem, a KeeperException will be thrown.
991   *
992   * @param zkw zk reference
993   * @param znode path of node
994   * @param data data of node
995   * @return true if node created, false if not, watch set in both cases
996   * @throws KeeperException if unexpected zookeeper exception
997   */
998  public static boolean createNodeIfNotExistsAndWatch(
999          ZKWatcher zkw, String znode, byte [] data)
1000    throws KeeperException {
1001    boolean ret = true;
1002    try {
1003      zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
1004          CreateMode.PERSISTENT);
1005    } catch (KeeperException.NodeExistsException nee) {
1006      ret = false;
1007    } catch (InterruptedException e) {
1008      zkw.interruptedException(e);
1009      return false;
1010    }
1011    try {
1012      zkw.getRecoverableZooKeeper().exists(znode, zkw);
1013    } catch (InterruptedException e) {
1014      zkw.interruptedException(e);
1015      return false;
1016    }
1017    return ret;
1018  }
1019
1020  /**
1021   * Creates the specified znode with the specified data but does not watch it.
1022   *
1023   * Returns the znode of the newly created node
1024   *
1025   * If there is another problem, a KeeperException will be thrown.
1026   *
1027   * @param zkw zk reference
1028   * @param znode path of node
1029   * @param data data of node
1030   * @param createMode specifying whether the node to be created is ephemeral and/or sequential
1031   * @return true name of the newly created znode or null
1032   * @throws KeeperException if unexpected zookeeper exception
1033   */
1034  public static String createNodeIfNotExistsNoWatch(ZKWatcher zkw, String znode, byte[] data,
1035      CreateMode createMode) throws KeeperException {
1036    String createdZNode = null;
1037    try {
1038      createdZNode = zkw.getRecoverableZooKeeper().create(znode, data,
1039          createACL(zkw, znode), createMode);
1040    } catch (KeeperException.NodeExistsException nee) {
1041      return znode;
1042    } catch (InterruptedException e) {
1043      zkw.interruptedException(e);
1044      return null;
1045    }
1046    return createdZNode;
1047  }
1048
1049  /**
1050   * Creates the specified node with the specified data and watches it.
1051   *
1052   * <p>Throws an exception if the node already exists.
1053   *
1054   * <p>The node created is persistent and open access.
1055   *
1056   * <p>Returns the version number of the created node if successful.
1057   *
1058   * @param zkw zk reference
1059   * @param znode path of node to create
1060   * @param data data of node to create
1061   * @return version of node created
1062   * @throws KeeperException if unexpected zookeeper exception
1063   * @throws KeeperException.NodeExistsException if node already exists
1064   */
1065  public static int createAndWatch(ZKWatcher zkw,
1066      String znode, byte [] data)
1067    throws KeeperException, KeeperException.NodeExistsException {
1068    try {
1069      zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
1070          CreateMode.PERSISTENT);
1071      Stat stat = zkw.getRecoverableZooKeeper().exists(znode, zkw);
1072      if (stat == null){
1073        // Likely a race condition. Someone deleted the znode.
1074        throw KeeperException.create(KeeperException.Code.SYSTEMERROR,
1075            "ZK.exists returned null (i.e.: znode does not exist) for znode=" + znode);
1076      }
1077
1078      return stat.getVersion();
1079    } catch (InterruptedException e) {
1080      zkw.interruptedException(e);
1081      return -1;
1082    }
1083  }
1084
1085  /**
1086   * Async creates the specified node with the specified data.
1087   *
1088   * <p>Throws an exception if the node already exists.
1089   *
1090   * <p>The node created is persistent and open access.
1091   *
1092   * @param zkw zk reference
1093   * @param znode path of node to create
1094   * @param data data of node to create
1095   * @param cb the callback to use for the creation
1096   * @param ctx the context to use for the creation
1097   */
1098  public static void asyncCreate(ZKWatcher zkw,
1099      String znode, byte [] data, final AsyncCallback.StringCallback cb,
1100      final Object ctx) {
1101    zkw.getRecoverableZooKeeper().getZooKeeper().create(znode, data,
1102        createACL(zkw, znode), CreateMode.PERSISTENT, cb, ctx);
1103  }
1104
1105  /**
1106   * Creates the specified node, iff the node does not exist.  Does not set a
1107   * watch and fails silently if the node already exists.
1108   *
1109   * The node created is persistent and open access.
1110   *
1111   * @param zkw zk reference
1112   * @param znode path of node
1113   * @throws KeeperException if unexpected zookeeper exception
1114   */
1115  public static void createAndFailSilent(ZKWatcher zkw,
1116      String znode) throws KeeperException {
1117    createAndFailSilent(zkw, znode, new byte[0]);
1118  }
1119
1120  /**
1121   * Creates the specified node containing specified data, iff the node does not exist.  Does
1122   * not set a watch and fails silently if the node already exists.
1123   *
1124   * The node created is persistent and open access.
1125   *
1126   * @param zkw zk reference
1127   * @param znode path of node
1128   * @param data a byte array data to store in the znode
1129   * @throws KeeperException if unexpected zookeeper exception
1130   */
1131  public static void createAndFailSilent(ZKWatcher zkw,
1132      String znode, byte[] data)
1133    throws KeeperException {
1134    createAndFailSilent(zkw,
1135        (CreateAndFailSilent)ZKUtilOp.createAndFailSilent(znode, data));
1136  }
1137
1138  private static void createAndFailSilent(ZKWatcher zkw, CreateAndFailSilent cafs)
1139    throws KeeperException {
1140    CreateRequest create = (CreateRequest)toZooKeeperOp(zkw, cafs).toRequestRecord();
1141    String znode = create.getPath();
1142    try {
1143      RecoverableZooKeeper zk = zkw.getRecoverableZooKeeper();
1144      if (zk.exists(znode, false) == null) {
1145        zk.create(znode, create.getData(), create.getAcl(), CreateMode.fromFlag(create.getFlags()));
1146      }
1147    } catch(KeeperException.NodeExistsException nee) {
1148    } catch(KeeperException.NoAuthException nee){
1149      try {
1150        if (null == zkw.getRecoverableZooKeeper().exists(znode, false)) {
1151          // If we failed to create the file and it does not already exist.
1152          throw(nee);
1153        }
1154      } catch (InterruptedException ie) {
1155        zkw.interruptedException(ie);
1156      }
1157    } catch(InterruptedException ie) {
1158      zkw.interruptedException(ie);
1159    }
1160  }
1161
1162  /**
1163   * Creates the specified node and all parent nodes required for it to exist.
1164   *
1165   * No watches are set and no errors are thrown if the node already exists.
1166   *
1167   * The nodes created are persistent and open access.
1168   *
1169   * @param zkw zk reference
1170   * @param znode path of node
1171   * @throws KeeperException if unexpected zookeeper exception
1172   */
1173  public static void createWithParents(ZKWatcher zkw, String znode)
1174    throws KeeperException {
1175    createWithParents(zkw, znode, new byte[0]);
1176  }
1177
1178  /**
1179   * Creates the specified node and all parent nodes required for it to exist.  The creation of
1180   * parent znodes is not atomic with the leafe znode creation but the data is written atomically
1181   * when the leaf node is created.
1182   *
1183   * No watches are set and no errors are thrown if the node already exists.
1184   *
1185   * The nodes created are persistent and open access.
1186   *
1187   * @param zkw zk reference
1188   * @param znode path of node
1189   * @throws KeeperException if unexpected zookeeper exception
1190   */
1191  public static void createWithParents(ZKWatcher zkw, String znode, byte[] data)
1192    throws KeeperException {
1193    try {
1194      if(znode == null) {
1195        return;
1196      }
1197      zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
1198          CreateMode.PERSISTENT);
1199    } catch(KeeperException.NodeExistsException nee) {
1200      return;
1201    } catch(KeeperException.NoNodeException nne) {
1202      createWithParents(zkw, getParent(znode));
1203      createWithParents(zkw, znode, data);
1204    } catch(InterruptedException ie) {
1205      zkw.interruptedException(ie);
1206    }
1207  }
1208
1209  //
1210  // Deletes
1211  //
1212
1213  /**
1214   * Delete the specified node.  Sets no watches.  Throws all exceptions.
1215   */
1216  public static void deleteNode(ZKWatcher zkw, String node)
1217    throws KeeperException {
1218    deleteNode(zkw, node, -1);
1219  }
1220
1221  /**
1222   * Delete the specified node with the specified version.  Sets no watches.
1223   * Throws all exceptions.
1224   */
1225  public static boolean deleteNode(ZKWatcher zkw, String node,
1226                                   int version)
1227    throws KeeperException {
1228    try {
1229      zkw.getRecoverableZooKeeper().delete(node, version);
1230      return true;
1231    } catch(KeeperException.BadVersionException bve) {
1232      return false;
1233    } catch(InterruptedException ie) {
1234      zkw.interruptedException(ie);
1235      return false;
1236    }
1237  }
1238
1239  /**
1240   * Deletes the specified node.  Fails silent if the node does not exist.
1241   *
1242   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
1243   * @param node the node to delete
1244   * @throws KeeperException if a ZooKeeper operation fails
1245   */
1246  public static void deleteNodeFailSilent(ZKWatcher zkw, String node)
1247    throws KeeperException {
1248    deleteNodeFailSilent(zkw,
1249      (DeleteNodeFailSilent)ZKUtilOp.deleteNodeFailSilent(node));
1250  }
1251
1252  private static void deleteNodeFailSilent(ZKWatcher zkw,
1253      DeleteNodeFailSilent dnfs) throws KeeperException {
1254    DeleteRequest delete = (DeleteRequest)toZooKeeperOp(zkw, dnfs).toRequestRecord();
1255    try {
1256      zkw.getRecoverableZooKeeper().delete(delete.getPath(), delete.getVersion());
1257    } catch(KeeperException.NoNodeException nne) {
1258    } catch(InterruptedException ie) {
1259      zkw.interruptedException(ie);
1260    }
1261  }
1262
1263
1264  /**
1265   * Delete the specified node and all of it's children.
1266   * <p>
1267   * If the node does not exist, just returns.
1268   * <p>
1269   * Sets no watches. Throws all exceptions besides dealing with deletion of
1270   * children.
1271   */
1272  public static void deleteNodeRecursively(ZKWatcher zkw, String node)
1273    throws KeeperException {
1274    deleteNodeRecursivelyMultiOrSequential(zkw, true, node);
1275  }
1276
1277  /**
1278   * Delete all the children of the specified node but not the node itself.
1279   *
1280   * Sets no watches.  Throws all exceptions besides dealing with deletion of
1281   * children.
1282   *
1283   * @throws KeeperException if a ZooKeeper operation fails
1284   */
1285  public static void deleteChildrenRecursively(ZKWatcher zkw, String node)
1286      throws KeeperException {
1287    deleteChildrenRecursivelyMultiOrSequential(zkw, true, node);
1288  }
1289
1290  /**
1291   * Delete all the children of the specified node but not the node itself. This
1292   * will first traverse the znode tree for listing the children and then delete
1293   * these znodes using multi-update api or sequential based on the specified
1294   * configurations.
1295   * <p>
1296   * Sets no watches. Throws all exceptions besides dealing with deletion of
1297   * children.
1298   * <p>
1299   * If the following is true:
1300   * <ul>
1301   * <li>runSequentialOnMultiFailure is true
1302   * </ul>
1303   * on calling multi, we get a ZooKeeper exception that can be handled by a
1304   * sequential call(*), we retry the operations one-by-one (sequentially).
1305   *
1306   * @param zkw
1307   *          - zk reference
1308   * @param runSequentialOnMultiFailure
1309   *          - if true when we get a ZooKeeper exception that could retry the
1310   *          operations one-by-one (sequentially)
1311   * @param pathRoots
1312   *          - path of the parent node(s)
1313   * @throws KeeperException.NotEmptyException
1314   *           if node has children while deleting
1315   * @throws KeeperException
1316   *           if unexpected ZooKeeper exception
1317   * @throws IllegalArgumentException
1318   *           if an invalid path is specified
1319   */
1320  public static void deleteChildrenRecursivelyMultiOrSequential(
1321          ZKWatcher zkw, boolean runSequentialOnMultiFailure,
1322          String... pathRoots) throws KeeperException {
1323    if (pathRoots == null || pathRoots.length <= 0) {
1324      LOG.warn("Given path is not valid!");
1325      return;
1326    }
1327    List<ZKUtilOp> ops = new ArrayList<>();
1328    for (String eachRoot : pathRoots) {
1329      List<String> children = listChildrenBFSNoWatch(zkw, eachRoot);
1330      // Delete the leaves first and eventually get rid of the root
1331      for (int i = children.size() - 1; i >= 0; --i) {
1332        ops.add(ZKUtilOp.deleteNodeFailSilent(children.get(i)));
1333      }
1334    }
1335    // atleast one element should exist
1336    if (ops.size() > 0) {
1337      multiOrSequential(zkw, ops, runSequentialOnMultiFailure);
1338    }
1339  }
1340
1341  /**
1342   * Delete the specified node and its children. This traverse the
1343   * znode tree for listing the children and then delete
1344   * these znodes including the parent using multi-update api or
1345   * sequential based on the specified configurations.
1346   * <p>
1347   * Sets no watches. Throws all exceptions besides dealing with deletion of
1348   * children.
1349   * <p>
1350   * If the following is true:
1351   * <ul>
1352   * <li>runSequentialOnMultiFailure is true
1353   * </ul>
1354   * on calling multi, we get a ZooKeeper exception that can be handled by a
1355   * sequential call(*), we retry the operations one-by-one (sequentially).
1356   *
1357   * @param zkw
1358   *          - zk reference
1359   * @param runSequentialOnMultiFailure
1360   *          - if true when we get a ZooKeeper exception that could retry the
1361   *          operations one-by-one (sequentially)
1362   * @param pathRoots
1363   *          - path of the parent node(s)
1364   * @throws KeeperException.NotEmptyException
1365   *           if node has children while deleting
1366   * @throws KeeperException
1367   *           if unexpected ZooKeeper exception
1368   * @throws IllegalArgumentException
1369   *           if an invalid path is specified
1370   */
1371  public static void deleteNodeRecursivelyMultiOrSequential(ZKWatcher zkw,
1372      boolean runSequentialOnMultiFailure, String... pathRoots) throws KeeperException {
1373    if (pathRoots == null || pathRoots.length <= 0) {
1374      LOG.warn("Given path is not valid!");
1375      return;
1376    }
1377    List<ZKUtilOp> ops = new ArrayList<>();
1378    for (String eachRoot : pathRoots) {
1379      // ZooKeeper Watches are one time triggers; When children of parent nodes are deleted
1380      // recursively, must set another watch, get notified of delete node
1381      List<String> children = listChildrenBFSAndWatchThem(zkw, eachRoot);
1382      // Delete the leaves first and eventually get rid of the root
1383      for (int i = children.size() - 1; i >= 0; --i) {
1384        ops.add(ZKUtilOp.deleteNodeFailSilent(children.get(i)));
1385      }
1386      try {
1387        if (zkw.getRecoverableZooKeeper().exists(eachRoot, zkw) != null) {
1388          ops.add(ZKUtilOp.deleteNodeFailSilent(eachRoot));
1389        }
1390      } catch (InterruptedException e) {
1391        zkw.interruptedException(e);
1392      }
1393    }
1394    // atleast one element should exist
1395    if (ops.size() > 0) {
1396      multiOrSequential(zkw, ops, runSequentialOnMultiFailure);
1397    }
1398  }
1399
1400  /**
1401   * BFS Traversal of all the children under path, with the entries in the list,
1402   * in the same order as that of the traversal. Lists all the children without
1403   * setting any watches.
1404   *
1405   * @param zkw
1406   *          - zk reference
1407   * @param znode
1408   *          - path of node
1409   * @return list of children znodes under the path
1410   * @throws KeeperException
1411   *           if unexpected ZooKeeper exception
1412   */
1413  private static List<String> listChildrenBFSNoWatch(ZKWatcher zkw,
1414      final String znode) throws KeeperException {
1415    Deque<String> queue = new LinkedList<>();
1416    List<String> tree = new ArrayList<>();
1417    queue.add(znode);
1418    while (true) {
1419      String node = queue.pollFirst();
1420      if (node == null) {
1421        break;
1422      }
1423      List<String> children = listChildrenNoWatch(zkw, node);
1424      if (children == null) {
1425        continue;
1426      }
1427      for (final String child : children) {
1428        final String childPath = node + "/" + child;
1429        queue.add(childPath);
1430        tree.add(childPath);
1431      }
1432    }
1433    return tree;
1434  }
1435
1436  /**
1437   * BFS Traversal of all the children under path, with the entries in the list,
1438   * in the same order as that of the traversal.
1439   * Lists all the children and set watches on to them.
1440   *
1441   * @param zkw
1442   *          - zk reference
1443   * @param znode
1444   *          - path of node
1445   * @return list of children znodes under the path
1446   * @throws KeeperException
1447   *           if unexpected ZooKeeper exception
1448   */
1449  private static List<String> listChildrenBFSAndWatchThem(ZKWatcher zkw, final String znode)
1450      throws KeeperException {
1451    Deque<String> queue = new LinkedList<>();
1452    List<String> tree = new ArrayList<>();
1453    queue.add(znode);
1454    while (true) {
1455      String node = queue.pollFirst();
1456      if (node == null) {
1457        break;
1458      }
1459      List<String> children = listChildrenAndWatchThem(zkw, node);
1460      if (children == null) {
1461        continue;
1462      }
1463      for (final String child : children) {
1464        final String childPath = node + "/" + child;
1465        queue.add(childPath);
1466        tree.add(childPath);
1467      }
1468    }
1469    return tree;
1470  }
1471
1472  /**
1473   * Represents an action taken by ZKUtil, e.g. createAndFailSilent.
1474   * These actions are higher-level than ZKOp actions, which represent
1475   * individual actions in the ZooKeeper API, like create.
1476   */
1477  public abstract static class ZKUtilOp {
1478    private String path;
1479
1480    private ZKUtilOp(String path) {
1481      this.path = path;
1482    }
1483
1484    /**
1485     * @return a createAndFailSilent ZKUtilOp
1486     */
1487    public static ZKUtilOp createAndFailSilent(String path, byte[] data) {
1488      return new CreateAndFailSilent(path, data);
1489    }
1490
1491    /**
1492     * @return a deleteNodeFailSilent ZKUtilOP
1493     */
1494    public static ZKUtilOp deleteNodeFailSilent(String path) {
1495      return new DeleteNodeFailSilent(path);
1496    }
1497
1498    /**
1499     * @return a setData ZKUtilOp
1500     */
1501    public static ZKUtilOp setData(String path, byte[] data) {
1502      return new SetData(path, data);
1503    }
1504
1505    /**
1506     * @return a setData ZKUtilOp
1507     */
1508    public static ZKUtilOp setData(String path, byte[] data, int version) {
1509      return new SetData(path, data, version);
1510    }
1511
1512    /**
1513     * @return path to znode where the ZKOp will occur
1514     */
1515    public String getPath() {
1516      return path;
1517    }
1518
1519    /**
1520     * ZKUtilOp representing createAndFailSilent in ZooKeeper
1521     * (attempt to create node, ignore error if already exists)
1522     */
1523    public static final class CreateAndFailSilent extends ZKUtilOp {
1524      private byte [] data;
1525
1526      private CreateAndFailSilent(String path, byte [] data) {
1527        super(path);
1528        this.data = data;
1529      }
1530
1531      public byte[] getData() {
1532        return data;
1533      }
1534
1535      @Override
1536      public boolean equals(Object o) {
1537        if (this == o) {
1538          return true;
1539        }
1540        if (!(o instanceof CreateAndFailSilent)) {
1541          return false;
1542        }
1543
1544        CreateAndFailSilent op = (CreateAndFailSilent) o;
1545        return getPath().equals(op.getPath()) && Arrays.equals(data, op.data);
1546      }
1547
1548      @Override
1549      public int hashCode() {
1550        int ret = 17 + getPath().hashCode() * 31;
1551        return ret * 31 + Bytes.hashCode(data);
1552      }
1553    }
1554
1555    /**
1556     * ZKUtilOp representing deleteNodeFailSilent in ZooKeeper
1557     * (attempt to delete node, ignore error if node doesn't exist)
1558     */
1559    public static final class DeleteNodeFailSilent extends ZKUtilOp {
1560      private DeleteNodeFailSilent(String path) {
1561        super(path);
1562      }
1563
1564      @Override
1565      public boolean equals(Object o) {
1566        if (this == o) {
1567          return true;
1568        }
1569        if (!(o instanceof DeleteNodeFailSilent)) {
1570          return false;
1571        }
1572
1573        return super.equals(o);
1574      }
1575
1576      @Override
1577      public int hashCode() {
1578        return getPath().hashCode();
1579      }
1580    }
1581
1582    /**
1583     * ZKUtilOp representing setData in ZooKeeper
1584     */
1585    public static final class SetData extends ZKUtilOp {
1586      private byte[] data;
1587      private int version = -1;
1588
1589      private SetData(String path, byte[] data) {
1590        super(path);
1591        this.data = data;
1592      }
1593
1594      private SetData(String path, byte[] data, int version) {
1595        super(path);
1596        this.data = data;
1597        this.version = version;
1598      }
1599
1600      public byte[] getData() {
1601        return data;
1602      }
1603
1604      public int getVersion() {
1605        return version;
1606      }
1607
1608      @Override
1609      public boolean equals(Object o) {
1610        if (this == o) {
1611          return true;
1612        }
1613        if (!(o instanceof SetData)) {
1614          return false;
1615        }
1616
1617        SetData op = (SetData) o;
1618        return getPath().equals(op.getPath()) && Arrays.equals(data, op.data)
1619            && getVersion() == op.getVersion();
1620      }
1621
1622      @Override
1623      public int hashCode() {
1624        int ret = getPath().hashCode();
1625        ret = ret * 31 + Bytes.hashCode(data);
1626        return ret * 31 + Integer.hashCode(version);
1627      }
1628    }
1629  }
1630
1631  /**
1632   * Convert from ZKUtilOp to ZKOp
1633   */
1634  private static Op toZooKeeperOp(ZKWatcher zkw, ZKUtilOp op) throws UnsupportedOperationException {
1635    if(op == null) {
1636      return null;
1637    }
1638
1639    if (op instanceof CreateAndFailSilent) {
1640      CreateAndFailSilent cafs = (CreateAndFailSilent)op;
1641      return Op.create(cafs.getPath(), cafs.getData(), createACL(zkw, cafs.getPath()),
1642        CreateMode.PERSISTENT);
1643    } else if (op instanceof DeleteNodeFailSilent) {
1644      DeleteNodeFailSilent dnfs = (DeleteNodeFailSilent)op;
1645      return Op.delete(dnfs.getPath(), -1);
1646    } else if (op instanceof SetData) {
1647      SetData sd = (SetData) op;
1648      return Op.setData(sd.getPath(), sd.getData(), sd.getVersion());
1649    } else {
1650      throw new UnsupportedOperationException("Unexpected ZKUtilOp type: "
1651        + op.getClass().getName());
1652    }
1653  }
1654
1655  /**
1656   * Use ZooKeeper's multi-update functionality.
1657   *
1658   * If all of the following are true:
1659   * - runSequentialOnMultiFailure is true
1660   * - on calling multi, we get a ZooKeeper exception that can be handled by a sequential call(*)
1661   * Then:
1662   * - we retry the operations one-by-one (sequentially)
1663   *
1664   * Note *: an example is receiving a NodeExistsException from a "create" call.  Without multi,
1665   * a user could call "createAndFailSilent" to ensure that a node exists if they don't care who
1666   * actually created the node (i.e. the NodeExistsException from ZooKeeper is caught).
1667   * This will cause all operations in the multi to fail, however, because
1668   * the NodeExistsException that zk.create throws will fail the multi transaction.
1669   * In this case, if the previous conditions hold, the commands are run sequentially, which should
1670   * result in the correct final state, but means that the operations will not run atomically.
1671   *
1672   * @throws KeeperException if a ZooKeeper operation fails
1673   */
1674  public static void multiOrSequential(ZKWatcher zkw, List<ZKUtilOp> ops,
1675                                       boolean runSequentialOnMultiFailure) throws KeeperException {
1676    if (zkw.getConfiguration().get("hbase.zookeeper.useMulti") != null) {
1677      LOG.warn("hbase.zookeeper.useMulti is deprecated. Default to true always.");
1678    }
1679    if (ops == null) {
1680      return;
1681    }
1682
1683    List<Op> zkOps = new LinkedList<>();
1684    for (ZKUtilOp op : ops) {
1685      zkOps.add(toZooKeeperOp(zkw, op));
1686    }
1687    try {
1688      zkw.getRecoverableZooKeeper().multi(zkOps);
1689    } catch (KeeperException ke) {
1690      switch (ke.code()) {
1691        case NODEEXISTS:
1692        case NONODE:
1693        case BADVERSION:
1694        case NOAUTH:
1695          // if we get an exception that could be solved by running sequentially
1696          // (and the client asked us to), then break out and run sequentially
1697          if (runSequentialOnMultiFailure) {
1698            LOG.info("On call to ZK.multi, received exception: " + ke.toString() + "."
1699                + "  Attempting to run operations sequentially because"
1700                + " runSequentialOnMultiFailure is: " + runSequentialOnMultiFailure + ".");
1701            processSequentially(zkw, ops);
1702            break;
1703          }
1704        default:
1705          throw ke;
1706      }
1707    } catch (InterruptedException ie) {
1708      zkw.interruptedException(ie);
1709    }
1710  }
1711
1712  private static void processSequentially(ZKWatcher zkw, List<ZKUtilOp> ops)
1713      throws KeeperException, NoNodeException {
1714    for (ZKUtilOp op : ops) {
1715      if (op instanceof CreateAndFailSilent) {
1716        createAndFailSilent(zkw, (CreateAndFailSilent) op);
1717      } else if (op instanceof DeleteNodeFailSilent) {
1718        deleteNodeFailSilent(zkw, (DeleteNodeFailSilent) op);
1719      } else if (op instanceof SetData) {
1720        setData(zkw, (SetData) op);
1721      } else {
1722        throw new UnsupportedOperationException("Unexpected ZKUtilOp type: "
1723            + op.getClass().getName());
1724      }
1725    }
1726  }
1727
1728  //
1729  // ZooKeeper cluster information
1730  //
1731
1732  /** @return String dump of everything in ZooKeeper. */
1733  public static String dump(ZKWatcher zkw) {
1734    StringBuilder sb = new StringBuilder();
1735    try {
1736      sb.append("HBase is rooted at ").append(zkw.getZNodePaths().baseZNode);
1737      sb.append("\nActive master address: ");
1738      try {
1739        sb.append(MasterAddressTracker.getMasterAddress(zkw));
1740      } catch (IOException e) {
1741        sb.append("<<FAILED LOOKUP: " + e.getMessage() + ">>");
1742      }
1743      sb.append("\nBackup master addresses:");
1744      final List<String> backupMasterChildrenNoWatchList = listChildrenNoWatch(zkw,
1745              zkw.getZNodePaths().backupMasterAddressesZNode);
1746      if (backupMasterChildrenNoWatchList != null) {
1747        for (String child : backupMasterChildrenNoWatchList) {
1748          sb.append("\n ").append(child);
1749        }
1750      }
1751      sb.append("\nRegion server holding hbase:meta: "
1752        + MetaTableLocator.getMetaRegionLocation(zkw));
1753      Configuration conf = HBaseConfiguration.create();
1754      int numMetaReplicas = conf.getInt(HConstants.META_REPLICAS_NUM,
1755               HConstants.DEFAULT_META_REPLICA_NUM);
1756      for (int i = 1; i < numMetaReplicas; i++) {
1757        sb.append("\nRegion server holding hbase:meta, replicaId " + i + " "
1758                    + MetaTableLocator.getMetaRegionLocation(zkw, i));
1759      }
1760      sb.append("\nRegion servers:");
1761      final List<String> rsChildrenNoWatchList =
1762              listChildrenNoWatch(zkw, zkw.getZNodePaths().rsZNode);
1763      if (rsChildrenNoWatchList != null) {
1764        for (String child : rsChildrenNoWatchList) {
1765          sb.append("\n ").append(child);
1766        }
1767      }
1768      try {
1769        getReplicationZnodesDump(zkw, sb);
1770      } catch (KeeperException ke) {
1771        LOG.warn("Couldn't get the replication znode dump", ke);
1772      }
1773      sb.append("\nQuorum Server Statistics:");
1774      String[] servers = zkw.getQuorum().split(",");
1775      for (String server : servers) {
1776        sb.append("\n ").append(server);
1777        try {
1778          String[] stat = getServerStats(server, ZKUtil.zkDumpConnectionTimeOut);
1779
1780          if (stat == null) {
1781            sb.append("[Error] invalid quorum server: " + server);
1782            break;
1783          }
1784
1785          for (String s : stat) {
1786            sb.append("\n  ").append(s);
1787          }
1788        } catch (Exception e) {
1789          sb.append("\n  ERROR: ").append(e.getMessage());
1790        }
1791      }
1792    } catch (KeeperException ke) {
1793      sb.append("\nFATAL ZooKeeper Exception!\n");
1794      sb.append("\n" + ke.getMessage());
1795    }
1796    return sb.toString();
1797  }
1798
1799  /**
1800   * Appends replication znodes to the passed StringBuilder.
1801   *
1802   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
1803   * @param sb the {@link StringBuilder} to append to
1804   * @throws KeeperException if a ZooKeeper operation fails
1805   */
1806  private static void getReplicationZnodesDump(ZKWatcher zkw, StringBuilder sb)
1807      throws KeeperException {
1808    String replicationZnode = zkw.getZNodePaths().replicationZNode;
1809
1810    if (ZKUtil.checkExists(zkw, replicationZnode) == -1) {
1811      return;
1812    }
1813
1814    // do a ls -r on this znode
1815    sb.append("\n").append(replicationZnode).append(": ");
1816    List<String> children = ZKUtil.listChildrenNoWatch(zkw, replicationZnode);
1817    if (children != null) {
1818      Collections.sort(children);
1819      for (String child : children) {
1820        String zNode = ZNodePaths.joinZNode(replicationZnode, child);
1821        if (zNode.equals(zkw.getZNodePaths().peersZNode)) {
1822          appendPeersZnodes(zkw, zNode, sb);
1823        } else if (zNode.equals(zkw.getZNodePaths().queuesZNode)) {
1824          appendRSZnodes(zkw, zNode, sb);
1825        } else if (zNode.equals(zkw.getZNodePaths().hfileRefsZNode)) {
1826          appendHFileRefsZNodes(zkw, zNode, sb);
1827        }
1828      }
1829    }
1830  }
1831
1832  private static void appendHFileRefsZNodes(ZKWatcher zkw, String hFileRefsZNode,
1833                                            StringBuilder sb) throws KeeperException {
1834    sb.append("\n").append(hFileRefsZNode).append(": ");
1835    final List<String> hFileRefChildrenNoWatchList =
1836            ZKUtil.listChildrenNoWatch(zkw, hFileRefsZNode);
1837    if (hFileRefChildrenNoWatchList != null) {
1838      for (String peerIdZNode : hFileRefChildrenNoWatchList) {
1839        String zNodeToProcess = ZNodePaths.joinZNode(hFileRefsZNode, peerIdZNode);
1840        sb.append("\n").append(zNodeToProcess).append(": ");
1841        List<String> peerHFileRefsZNodes = ZKUtil.listChildrenNoWatch(zkw, zNodeToProcess);
1842        if (peerHFileRefsZNodes != null) {
1843          sb.append(String.join(", ", peerHFileRefsZNodes));
1844        }
1845      }
1846    }
1847  }
1848
1849  /**
1850   * Returns a string with replication znodes and position of the replication log
1851   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
1852   * @return aq string of replication znodes and log positions
1853   */
1854  public static String getReplicationZnodesDump(ZKWatcher zkw) throws KeeperException {
1855    StringBuilder sb = new StringBuilder();
1856    getReplicationZnodesDump(zkw, sb);
1857    return sb.toString();
1858  }
1859
1860  private static void appendRSZnodes(ZKWatcher zkw, String znode, StringBuilder sb)
1861      throws KeeperException {
1862    List<String> stack = new LinkedList<>();
1863    stack.add(znode);
1864    do {
1865      String znodeToProcess = stack.remove(stack.size() - 1);
1866      sb.append("\n").append(znodeToProcess).append(": ");
1867      byte[] data;
1868      try {
1869        data = ZKUtil.getData(zkw, znodeToProcess);
1870      } catch (InterruptedException e) {
1871        zkw.interruptedException(e);
1872        return;
1873      }
1874      if (data != null && data.length > 0) { // log position
1875        long position = 0;
1876        try {
1877          position = ZKUtil.parseWALPositionFrom(ZKUtil.getData(zkw, znodeToProcess));
1878          sb.append(position);
1879        } catch (DeserializationException ignored) {
1880        } catch (InterruptedException e) {
1881          zkw.interruptedException(e);
1882          return;
1883        }
1884      }
1885      for (String zNodeChild : ZKUtil.listChildrenNoWatch(zkw, znodeToProcess)) {
1886        stack.add(ZNodePaths.joinZNode(znodeToProcess, zNodeChild));
1887      }
1888    } while (stack.size() > 0);
1889  }
1890
1891  private static void appendPeersZnodes(ZKWatcher zkw, String peersZnode,
1892                                        StringBuilder sb) throws KeeperException {
1893    int pblen = ProtobufUtil.lengthOfPBMagic();
1894    sb.append("\n").append(peersZnode).append(": ");
1895    for (String peerIdZnode : ZKUtil.listChildrenNoWatch(zkw, peersZnode)) {
1896      String znodeToProcess = ZNodePaths.joinZNode(peersZnode, peerIdZnode);
1897      byte[] data;
1898      try {
1899        data = ZKUtil.getData(zkw, znodeToProcess);
1900      } catch (InterruptedException e) {
1901        zkw.interruptedException(e);
1902        return;
1903      }
1904      // parse the data of the above peer znode.
1905      try {
1906        ReplicationProtos.ReplicationPeer.Builder builder =
1907          ReplicationProtos.ReplicationPeer.newBuilder();
1908        ProtobufUtil.mergeFrom(builder, data, pblen, data.length - pblen);
1909        String clusterKey = builder.getClusterkey();
1910        sb.append("\n").append(znodeToProcess).append(": ").append(clusterKey);
1911        // add the peer-state.
1912        appendPeerState(zkw, znodeToProcess, sb);
1913      } catch (IOException ipbe) {
1914        LOG.warn("Got Exception while parsing peer: " + znodeToProcess, ipbe);
1915      }
1916    }
1917  }
1918
1919  private static void appendPeerState(ZKWatcher zkw, String znodeToProcess, StringBuilder sb)
1920          throws KeeperException, InvalidProtocolBufferException {
1921    String peerState = zkw.getConfiguration().get("zookeeper.znode.replication.peers.state",
1922      "peer-state");
1923    int pblen = ProtobufUtil.lengthOfPBMagic();
1924    for (String child : ZKUtil.listChildrenNoWatch(zkw, znodeToProcess)) {
1925      if (!child.equals(peerState)) {
1926        continue;
1927      }
1928
1929      String peerStateZnode = ZNodePaths.joinZNode(znodeToProcess, child);
1930      sb.append("\n").append(peerStateZnode).append(": ");
1931      byte[] peerStateData;
1932      try {
1933        peerStateData = ZKUtil.getData(zkw, peerStateZnode);
1934        ReplicationProtos.ReplicationState.Builder builder =
1935            ReplicationProtos.ReplicationState.newBuilder();
1936        ProtobufUtil.mergeFrom(builder, peerStateData, pblen, peerStateData.length - pblen);
1937        sb.append(builder.getState().name());
1938      } catch (IOException ipbe) {
1939        LOG.warn("Got Exception while parsing peer: " + znodeToProcess, ipbe);
1940      } catch (InterruptedException e) {
1941        zkw.interruptedException(e);
1942        return;
1943      }
1944    }
1945  }
1946
1947  /**
1948   * Gets the statistics from the given server.
1949   *
1950   * @param server  The server to get the statistics from.
1951   * @param timeout  The socket timeout to use.
1952   * @return The array of response strings.
1953   * @throws IOException When the socket communication fails.
1954   */
1955  private static String[] getServerStats(String server, int timeout)
1956    throws IOException {
1957    String[] sp = server.split(":");
1958    if (sp.length == 0) {
1959      return null;
1960    }
1961
1962    String host = sp[0];
1963    int port = sp.length > 1 ? Integer.parseInt(sp[1])
1964        : HConstants.DEFAULT_ZOOKEEPER_CLIENT_PORT;
1965
1966    InetSocketAddress sockAddr = new InetSocketAddress(host, port);
1967    try (Socket socket = new Socket()) {
1968      socket.connect(sockAddr, timeout);
1969
1970      socket.setSoTimeout(timeout);
1971      try (PrintWriter out = new PrintWriter(new BufferedWriter(
1972          new OutputStreamWriter(socket.getOutputStream(), StandardCharsets.UTF_8)), true);
1973          BufferedReader in = new BufferedReader(
1974              new InputStreamReader(socket.getInputStream(), StandardCharsets.UTF_8))) {
1975        out.println("stat");
1976        out.flush();
1977        ArrayList<String> res = new ArrayList<>();
1978        while (true) {
1979          String line = in.readLine();
1980          if (line != null) {
1981            res.add(line);
1982          } else {
1983            break;
1984          }
1985        }
1986        return res.toArray(new String[res.size()]);
1987      }
1988    }
1989  }
1990
1991  private static void logRetrievedMsg(final ZKWatcher zkw,
1992      final String znode, final byte [] data, final boolean watcherSet) {
1993    if (!LOG.isTraceEnabled()) {
1994      return;
1995    }
1996
1997    LOG.trace(zkw.prefix("Retrieved " + ((data == null)? 0: data.length) +
1998      " byte(s) of data from znode " + znode +
1999      (watcherSet? " and set watcher; ": "; data=") +
2000      (data == null? "null": data.length == 0? "empty": (
2001          znode.startsWith(zkw.getZNodePaths().metaZNodePrefix)?
2002            getServerNameOrEmptyString(data):
2003          znode.startsWith(zkw.getZNodePaths().backupMasterAddressesZNode)?
2004            getServerNameOrEmptyString(data):
2005          StringUtils.abbreviate(Bytes.toStringBinary(data), 32)))));
2006  }
2007
2008  private static String getServerNameOrEmptyString(final byte [] data) {
2009    try {
2010      return ProtobufUtil.parseServerNameFrom(data).toString();
2011    } catch (DeserializationException e) {
2012      return "";
2013    }
2014  }
2015
2016  /**
2017   * Waits for HBase installation's base (parent) znode to become available.
2018   * @throws IOException on ZK errors
2019   */
2020  public static void waitForBaseZNode(Configuration conf) throws IOException {
2021    LOG.info("Waiting until the base znode is available");
2022    String parentZNode = conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
2023        HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
2024    ZooKeeper zk = new ZooKeeper(ZKConfig.getZKQuorumServersString(conf),
2025        conf.getInt(HConstants.ZK_SESSION_TIMEOUT,
2026        HConstants.DEFAULT_ZK_SESSION_TIMEOUT), EmptyWatcher.instance);
2027
2028    final int maxTimeMs = 10000;
2029    final int maxNumAttempts = maxTimeMs / HConstants.SOCKET_RETRY_WAIT_MS;
2030
2031    KeeperException keeperEx = null;
2032    try {
2033      try {
2034        for (int attempt = 0; attempt < maxNumAttempts; ++attempt) {
2035          try {
2036            if (zk.exists(parentZNode, false) != null) {
2037              LOG.info("Parent znode exists: {}", parentZNode);
2038              keeperEx = null;
2039              break;
2040            }
2041          } catch (KeeperException e) {
2042            keeperEx = e;
2043          }
2044          Threads.sleepWithoutInterrupt(HConstants.SOCKET_RETRY_WAIT_MS);
2045        }
2046      } finally {
2047        zk.close();
2048      }
2049    } catch (InterruptedException ex) {
2050      Thread.currentThread().interrupt();
2051    }
2052
2053    if (keeperEx != null) {
2054      throw new IOException(keeperEx);
2055    }
2056  }
2057
2058  /**
2059   * Convert a {@link DeserializationException} to a more palatable {@link KeeperException}.
2060   * Used when can't let a {@link DeserializationException} out w/o changing public API.
2061   * @param e Exception to convert
2062   * @return Converted exception
2063   */
2064  public static KeeperException convert(final DeserializationException e) {
2065    KeeperException ke = new KeeperException.DataInconsistencyException();
2066    ke.initCause(e);
2067    return ke;
2068  }
2069
2070  /**
2071   * Recursively print the current state of ZK (non-transactional)
2072   * @param root name of the root directory in zk to print
2073   */
2074  public static void logZKTree(ZKWatcher zkw, String root) {
2075    if (!LOG.isDebugEnabled()) {
2076      return;
2077    }
2078
2079    LOG.debug("Current zk system:");
2080    String prefix = "|-";
2081    LOG.debug(prefix + root);
2082    try {
2083      logZKTree(zkw, root, prefix);
2084    } catch (KeeperException e) {
2085      throw new RuntimeException(e);
2086    }
2087  }
2088
2089  /**
2090   * Helper method to print the current state of the ZK tree.
2091   * @see #logZKTree(ZKWatcher, String)
2092   * @throws KeeperException if an unexpected exception occurs
2093   */
2094  private static void logZKTree(ZKWatcher zkw, String root, String prefix)
2095      throws KeeperException {
2096    List<String> children = ZKUtil.listChildrenNoWatch(zkw, root);
2097
2098    if (children == null) {
2099      return;
2100    }
2101
2102    for (String child : children) {
2103      LOG.debug(prefix + child);
2104      String node = ZNodePaths.joinZNode(root.equals("/") ? "" : root, child);
2105      logZKTree(zkw, node, prefix + "---");
2106    }
2107  }
2108
2109  /**
2110   * @param position the position to serialize
2111   * @return Serialized protobuf of <code>position</code> with pb magic prefix prepended suitable
2112   *         for use as content of an wal position in a replication queue.
2113   */
2114  public static byte[] positionToByteArray(final long position) {
2115    byte[] bytes = ReplicationProtos.ReplicationHLogPosition.newBuilder().setPosition(position)
2116        .build().toByteArray();
2117    return ProtobufUtil.prependPBMagic(bytes);
2118  }
2119
2120  /**
2121   * @param bytes - Content of a WAL position znode.
2122   * @return long - The current WAL position.
2123   * @throws DeserializationException if the WAL position cannot be parsed
2124   */
2125  public static long parseWALPositionFrom(final byte[] bytes) throws DeserializationException {
2126    if (bytes == null) {
2127      throw new DeserializationException("Unable to parse null WAL position.");
2128    }
2129    if (ProtobufUtil.isPBMagicPrefix(bytes)) {
2130      int pblen = ProtobufUtil.lengthOfPBMagic();
2131      ReplicationProtos.ReplicationHLogPosition.Builder builder =
2132          ReplicationProtos.ReplicationHLogPosition.newBuilder();
2133      ReplicationProtos.ReplicationHLogPosition position;
2134      try {
2135        ProtobufUtil.mergeFrom(builder, bytes, pblen, bytes.length - pblen);
2136        position = builder.build();
2137      } catch (IOException e) {
2138        throw new DeserializationException(e);
2139      }
2140      return position.getPosition();
2141    } else {
2142      if (bytes.length > 0) {
2143        return Bytes.toLong(bytes);
2144      }
2145      return 0;
2146    }
2147  }
2148}