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_META_REPLICA_NUM; 021import static org.apache.hadoop.hbase.HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT; 022import static org.apache.hadoop.hbase.HConstants.META_REPLICAS_NUM; 023import static org.apache.hadoop.hbase.HConstants.SPLIT_LOGDIR_NAME; 024import static org.apache.hadoop.hbase.HConstants.ZOOKEEPER_ZNODE_PARENT; 025import static org.apache.hadoop.hbase.client.RegionInfo.DEFAULT_REPLICA_ID; 026 027import java.util.Optional; 028import java.util.stream.IntStream; 029import org.apache.hadoop.conf.Configuration; 030import org.apache.hadoop.hbase.client.RegionInfo; 031import org.apache.yetus.audience.InterfaceAudience; 032 033import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableMap; 034 035/** 036 * Class that hold all the paths of znode for HBase. 037 */ 038@InterfaceAudience.Private 039public class ZNodePaths { 040 // TODO: Replace this with ZooKeeper constant when ZOOKEEPER-277 is resolved. 041 public static final char ZNODE_PATH_SEPARATOR = '/'; 042 043 public final static String META_ZNODE_PREFIX = "meta-region-server"; 044 045 // base znode for this cluster 046 public final String baseZNode; 047 // the prefix of meta znode, does not include baseZNode. 048 public final String metaZNodePrefix; 049 // znodes containing the locations of the servers hosting the meta replicas 050 public final ImmutableMap<Integer, String> metaReplicaZNodes; 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 public final String balancerZNode; 073 // znode containing the state of region normalizer 074 public final String regionNormalizerZNode; 075 // znode containing the state of all switches, currently there are split and merge child node. 076 public final String switchZNode; 077 // znode containing namespace descriptors 078 public final String namespaceZNode; 079 // znode of indicating master maintenance mode 080 public final String masterMaintZNode; 081 082 // znode containing all replication state. 083 public final String replicationZNode; 084 // znode containing a list of all remote slave (i.e. peer) clusters. 085 public final String peersZNode; 086 // znode containing all replication queues 087 public final String queuesZNode; 088 // znode containing queues of hfile references to be replicated 089 public final String hfileRefsZNode; 090 091 public ZNodePaths(Configuration conf) { 092 baseZNode = conf.get(ZOOKEEPER_ZNODE_PARENT, DEFAULT_ZOOKEEPER_ZNODE_PARENT); 093 ImmutableMap.Builder<Integer, String> builder = ImmutableMap.builder(); 094 metaZNodePrefix = conf.get("zookeeper.znode.metaserver", META_ZNODE_PREFIX); 095 String defaultMetaReplicaZNode = ZNodePaths.joinZNode(baseZNode, metaZNodePrefix); 096 builder.put(DEFAULT_REPLICA_ID, defaultMetaReplicaZNode); 097 int numMetaReplicas = conf.getInt(META_REPLICAS_NUM, DEFAULT_META_REPLICA_NUM); 098 IntStream.range(1, numMetaReplicas) 099 .forEachOrdered(i -> builder.put(i, defaultMetaReplicaZNode + "-" + i)); 100 metaReplicaZNodes = builder.build(); 101 rsZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.rs", "rs")); 102 drainingZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.draining.rs", "draining")); 103 masterAddressZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.master", "master")); 104 backupMasterAddressesZNode = 105 joinZNode(baseZNode, conf.get("zookeeper.znode.backup.masters", "backup-masters")); 106 clusterStateZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.state", "running")); 107 tableZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.tableEnableDisable", "table")); 108 clusterIdZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.clusterId", "hbaseid")); 109 splitLogZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.splitlog", SPLIT_LOGDIR_NAME)); 110 balancerZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.balancer", "balancer")); 111 regionNormalizerZNode = 112 joinZNode(baseZNode, conf.get("zookeeper.znode.regionNormalizer", "normalizer")); 113 switchZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.switch", "switch")); 114 namespaceZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.namespace", "namespace")); 115 masterMaintZNode = 116 joinZNode(baseZNode, conf.get("zookeeper.znode.masterMaintenance", "master-maintenance")); 117 replicationZNode = joinZNode(baseZNode, conf.get("zookeeper.znode.replication", "replication")); 118 peersZNode = 119 joinZNode(replicationZNode, conf.get("zookeeper.znode.replication.peers", "peers")); 120 queuesZNode = joinZNode(replicationZNode, conf.get("zookeeper.znode.replication.rs", "rs")); 121 hfileRefsZNode = joinZNode(replicationZNode, 122 conf.get("zookeeper.znode.replication.hfile.refs", "hfile-refs")); 123 } 124 125 @Override 126 public String toString() { 127 return new StringBuilder() 128 .append("ZNodePaths [baseZNode=").append(baseZNode) 129 .append(", metaReplicaZNodes=").append(metaReplicaZNodes) 130 .append(", rsZNode=").append(rsZNode) 131 .append(", drainingZNode=").append(drainingZNode) 132 .append(", masterAddressZNode=").append(masterAddressZNode) 133 .append(", backupMasterAddressesZNode=").append(backupMasterAddressesZNode) 134 .append(", clusterStateZNode=").append(clusterStateZNode) 135 .append(", tableZNode=").append(tableZNode) 136 .append(", clusterIdZNode=").append(clusterIdZNode) 137 .append(", splitLogZNode=").append(splitLogZNode) 138 .append(", balancerZNode=").append(balancerZNode) 139 .append(", regionNormalizerZNode=").append(regionNormalizerZNode) 140 .append(", switchZNode=").append(switchZNode) 141 .append(", namespaceZNode=").append(namespaceZNode) 142 .append(", masterMaintZNode=").append(masterMaintZNode) 143 .append(", replicationZNode=").append(replicationZNode) 144 .append(", peersZNode=").append(peersZNode) 145 .append(", queuesZNode=").append(queuesZNode) 146 .append(", hfileRefsZNode=").append(hfileRefsZNode) 147 .append("]").toString(); 148 } 149 150 /** 151 * Is the znode of any meta replica 152 * @param node 153 * @return true or false 154 */ 155 public boolean isAnyMetaReplicaZNode(String node) { 156 if (metaReplicaZNodes.containsValue(node)) { 157 return true; 158 } 159 return false; 160 } 161 162 /** 163 * Get the znode string corresponding to a replicaId 164 * @param replicaId 165 * @return znode 166 */ 167 public String getZNodeForReplica(int replicaId) { 168 // return a newly created path but don't update the cache of paths 169 // This is mostly needed for tests that attempt to create meta replicas 170 // from outside the master 171 return Optional.ofNullable(metaReplicaZNodes.get(replicaId)) 172 .orElseGet(() -> metaReplicaZNodes.get(DEFAULT_REPLICA_ID) + "-" + replicaId); 173 } 174 175 /** 176 * Parse the meta replicaId from the passed znode 177 * @param znode the name of the znode, does not include baseZNode 178 * @return replicaId 179 */ 180 public int getMetaReplicaIdFromZnode(String znode) { 181 if (znode.equals(metaZNodePrefix)) { 182 return RegionInfo.DEFAULT_REPLICA_ID; 183 } 184 return Integer.parseInt(znode.substring(metaZNodePrefix.length() + 1)); 185 } 186 187 /** 188 * Is it the default meta replica's znode 189 * @param znode the name of the znode, does not include baseZNode 190 * @return true or false 191 */ 192 public boolean isDefaultMetaReplicaZnode(String znode) { 193 return metaReplicaZNodes.get(DEFAULT_REPLICA_ID).equals(znode); 194 } 195 196 /** 197 * Returns whether the znode is supposed to be readable by the client and DOES NOT contain 198 * sensitive information (world readable). 199 */ 200 public boolean isClientReadable(String node) { 201 // Developer notice: These znodes are world readable. DO NOT add more znodes here UNLESS 202 // all clients need to access this data to work. Using zk for sharing data to clients (other 203 // than service lookup case is not a recommended design pattern. 204 return node.equals(baseZNode) || isAnyMetaReplicaZNode(node) || 205 node.equals(masterAddressZNode) || node.equals(clusterIdZNode) || node.equals(rsZNode) || 206 // /hbase/table and /hbase/table/foo is allowed, /hbase/table-lock is not 207 node.equals(tableZNode) || node.startsWith(tableZNode + "/"); 208 } 209 210 /** 211 * Join the prefix znode name with the suffix znode name to generate a proper full znode name. 212 * <p> 213 * Assumes prefix does not end with slash and suffix does not begin with it. 214 * @param prefix beginning of znode name 215 * @param suffix ending of znode name 216 * @return result of properly joining prefix with suffix 217 */ 218 public static String joinZNode(String prefix, String suffix) { 219 return prefix + ZNodePaths.ZNODE_PATH_SEPARATOR + suffix; 220 } 221}