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