001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.zookeeper;
019
020import static org.apache.hadoop.hbase.HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT;
021import static org.apache.hadoop.hbase.HConstants.SPLIT_LOGDIR_NAME;
022import static org.apache.hadoop.hbase.HConstants.ZOOKEEPER_ZNODE_PARENT;
023
024import org.apache.hadoop.conf.Configuration;
025import org.apache.hadoop.hbase.ServerName;
026import org.apache.hadoop.hbase.client.RegionInfo;
027import org.apache.hadoop.hbase.client.RegionReplicaUtil;
028import org.apache.yetus.audience.InterfaceAudience;
029
030/**
031 * Class that hold all the paths of znode for HBase.
032 */
033@InterfaceAudience.Private
034public class ZNodePaths {
035  // TODO: Replace this with ZooKeeper constant when ZOOKEEPER-277 is resolved.
036  public static final char ZNODE_PATH_SEPARATOR = '/';
037
038  public static final String META_ZNODE_PREFIX_CONF_KEY = "zookeeper.znode.metaserver";
039  public static final String META_ZNODE_PREFIX = "meta-region-server";
040  private static final String DEFAULT_SNAPSHOT_CLEANUP_ZNODE = "snapshot-cleanup";
041
042  // base znode for this cluster
043  public final String baseZNode;
044
045  /**
046   * The prefix of meta znode. Does not include baseZNode. Its a 'prefix' because meta replica id
047   * integer can be tagged on the end (if no number present, it is 'default' replica).
048   */
049  private final String metaZNodePrefix;
050
051  // znode containing ephemeral nodes of the regionservers
052  public final String rsZNode;
053  // znode containing ephemeral nodes of the draining regionservers
054  public final String drainingZNode;
055  // znode of currently active master
056  public final String masterAddressZNode;
057  // znode of this master in backup master directory, if not the active master
058  public final String backupMasterAddressesZNode;
059  // znode containing the current cluster state
060  public final String clusterStateZNode;
061  // znode used for table disabling/enabling
062  // Still used in hbase2 by MirroringTableStateManager; it mirrors internal table state out to
063  // zookeeper for hbase1 clients to make use of. If no hbase1 clients disable. See
064  // MirroringTableStateManager. To be removed in hbase3.
065  @Deprecated
066  public final String tableZNode;
067  // znode containing the unique cluster ID
068  public final String clusterIdZNode;
069  // znode used for log splitting work assignment
070  public final String splitLogZNode;
071  // znode containing the state of the load balancer
072  /**
073   * @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
074   *             state.
075   */
076  @Deprecated
077  public final String balancerZNode;
078  // znode containing the state of region normalizer
079  /**
080   * @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
081   *             state.
082   */
083  @Deprecated
084  public final String regionNormalizerZNode;
085  // znode containing the state of all switches, currently there are split and merge child node.
086  /**
087   * @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
088   *             state.
089   */
090  @Deprecated
091  public final String switchZNode;
092  // znode of indicating master maintenance mode
093  public final String masterMaintZNode;
094
095  // znode containing all replication state.
096  public final String replicationZNode;
097  // znode containing a list of all remote slave (i.e. peer) clusters.
098  public final String peersZNode;
099  // znode containing all replication queues
100  public final String queuesZNode;
101  // znode containing queues of hfile references to be replicated
102  public final String hfileRefsZNode;
103  // znode containing the state of the snapshot auto-cleanup
104  /**
105   * @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
106   *             state.
107   */
108  @Deprecated
109  public final String snapshotCleanupZNode;
110
111  public ZNodePaths(Configuration conf) {
112    baseZNode = conf.get(ZOOKEEPER_ZNODE_PARENT, DEFAULT_ZOOKEEPER_ZNODE_PARENT);
113    metaZNodePrefix = conf.get(META_ZNODE_PREFIX_CONF_KEY, META_ZNODE_PREFIX);
114    rsZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.rs", "rs"));
115    drainingZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.draining.rs", "draining"));
116    masterAddressZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.master", "master"));
117    backupMasterAddressesZNode =
118      joinZNode(baseZNode, conf.get("zookeeper.znode.backup.masters", "backup-masters"));
119    clusterStateZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.state", "running"));
120    tableZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.tableEnableDisable", "table"));
121    clusterIdZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.clusterId", "hbaseid"));
122    splitLogZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.splitlog", SPLIT_LOGDIR_NAME));
123    balancerZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.balancer", "balancer"));
124    regionNormalizerZNode =
125      joinZNode(baseZNode, conf.get("zookeeper.znode.regionNormalizer", "normalizer"));
126    switchZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.switch", "switch"));
127    masterMaintZNode =
128      joinZNode(baseZNode, conf.get("zookeeper.znode.masterMaintenance", "master-maintenance"));
129    replicationZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.replication", "replication"));
130    peersZNode =
131      joinZNode(replicationZNode, conf.get("zookeeper.znode.replication.peers", "peers"));
132    queuesZNode = joinZNode(replicationZNode, conf.get("zookeeper.znode.replication.rs", "rs"));
133    hfileRefsZNode =
134      joinZNode(replicationZNode, conf.get("zookeeper.znode.replication.hfile.refs", "hfile-refs"));
135    snapshotCleanupZNode = joinZNode(baseZNode,
136      conf.get("zookeeper.znode.snapshot.cleanup", DEFAULT_SNAPSHOT_CLEANUP_ZNODE));
137  }
138
139  @Override
140  public String toString() {
141    return new StringBuilder().append("ZNodePaths [baseZNode=").append(baseZNode)
142      .append(", rsZNode=").append(rsZNode).append(", drainingZNode=").append(drainingZNode)
143      .append(", masterAddressZNode=").append(masterAddressZNode)
144      .append(", backupMasterAddressesZNode=").append(backupMasterAddressesZNode)
145      .append(", clusterStateZNode=").append(clusterStateZNode).append(", tableZNode=")
146      .append(tableZNode).append(", clusterIdZNode=").append(clusterIdZNode)
147      .append(", splitLogZNode=").append(splitLogZNode).append(", balancerZNode=")
148      .append(balancerZNode).append(", regionNormalizerZNode=").append(regionNormalizerZNode)
149      .append(", switchZNode=").append(switchZNode).append(", masterMaintZNode=")
150      .append(masterMaintZNode).append(", replicationZNode=").append(replicationZNode)
151      .append(", peersZNode=").append(peersZNode).append(", queuesZNode=").append(queuesZNode)
152      .append(", hfileRefsZNode=").append(hfileRefsZNode).append(", snapshotCleanupZNode=")
153      .append(snapshotCleanupZNode).append("]").toString();
154  }
155
156  /** Returns the znode string corresponding to a replicaId */
157  public String getZNodeForReplica(int replicaId) {
158    if (RegionReplicaUtil.isDefaultReplica(replicaId)) {
159      return joinZNode(baseZNode, metaZNodePrefix);
160    } else {
161      return joinZNode(baseZNode, metaZNodePrefix + "-" + replicaId);
162    }
163  }
164
165  /**
166   * Parses the meta replicaId from the passed path.
167   * @param path the name of the full path which includes baseZNode.
168   */
169  public int getMetaReplicaIdFromPath(String path) {
170    // Extract the znode from path. The prefix is of the following format.
171    // baseZNode + PATH_SEPARATOR.
172    int prefixLen = baseZNode.length() + 1;
173    return getMetaReplicaIdFromZNode(path.substring(prefixLen));
174  }
175
176  /**
177   * Parse the meta replicaId from the passed znode
178   * @param znode the name of the znode, does not include baseZNode
179   */
180  public int getMetaReplicaIdFromZNode(String znode) {
181    return znode.equals(metaZNodePrefix)
182      ? RegionInfo.DEFAULT_REPLICA_ID
183      : Integer.parseInt(znode.substring(metaZNodePrefix.length() + 1));
184  }
185
186  /** Returns True if meta znode. */
187  public boolean isMetaZNodePrefix(String znode) {
188    return znode != null && znode.startsWith(this.metaZNodePrefix);
189  }
190
191  /** Returns True is the fully qualified path is for meta location */
192  public boolean isMetaZNodePath(String path) {
193    int prefixLen = baseZNode.length() + 1;
194    return path.length() > prefixLen && isMetaZNodePrefix(path.substring(prefixLen));
195  }
196
197  /**
198   * Returns whether the path is supposed to be readable by the client and DOES NOT contain
199   * sensitive information (world readable).
200   */
201  public boolean isClientReadable(String path) {
202    // Developer notice: These znodes are world readable. DO NOT add more znodes here UNLESS
203    // all clients need to access this data to work. Using zk for sharing data to clients (other
204    // than service lookup case is not a recommended design pattern.
205    return path.equals(baseZNode) || isMetaZNodePath(path) || path.equals(masterAddressZNode)
206      || path.equals(clusterIdZNode) || path.equals(rsZNode) ||
207      // /hbase/table and /hbase/table/foo is allowed, /hbase/table-lock is not
208      path.equals(tableZNode) || path.startsWith(tableZNode + "/");
209  }
210
211  public String getRsPath(ServerName sn) {
212    return joinZNode(rsZNode, sn.toString());
213  }
214
215  /**
216   * Join the prefix znode name with the suffix znode name to generate a proper full znode name.
217   * <p>
218   * Assumes prefix does not end with slash and suffix does not begin with it.
219   * @param prefix beginning of znode name
220   * @param suffix ending of znode name
221   * @return result of properly joining prefix with suffix
222   */
223  public static String joinZNode(String prefix, String... suffix) {
224    StringBuilder sb = new StringBuilder(prefix);
225    for (String s : suffix) {
226      sb.append(ZNodePaths.ZNODE_PATH_SEPARATOR).append(s);
227    }
228    return sb.toString();
229  }
230}