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.namespace; 019 020import java.io.IOException; 021import java.util.List; 022import java.util.concurrent.ConcurrentHashMap; 023import java.util.concurrent.ConcurrentMap; 024import org.apache.hadoop.hbase.MetaTableAccessor; 025import org.apache.hadoop.hbase.NamespaceDescriptor; 026import org.apache.hadoop.hbase.TableName; 027import org.apache.hadoop.hbase.client.RegionInfo; 028import org.apache.hadoop.hbase.master.MasterServices; 029import org.apache.hadoop.hbase.master.TableNamespaceManager; 030import org.apache.hadoop.hbase.quotas.QuotaExceededException; 031import org.apache.hadoop.hbase.util.Bytes; 032import org.apache.yetus.audience.InterfaceAudience; 033import org.slf4j.Logger; 034import org.slf4j.LoggerFactory; 035 036/** 037 * NamespaceStateManager manages state (in terms of quota) of all the namespaces. It contains a 038 * cache which is updated based on the hooks in the NamespaceAuditor class. 039 */ 040@InterfaceAudience.Private 041class NamespaceStateManager { 042 043 private static final Logger LOG = LoggerFactory.getLogger(NamespaceStateManager.class); 044 private ConcurrentMap<String, NamespaceTableAndRegionInfo> nsStateCache; 045 private MasterServices master; 046 private volatile boolean initialized = false; 047 048 public NamespaceStateManager(MasterServices masterServices) { 049 nsStateCache = new ConcurrentHashMap<>(); 050 master = masterServices; 051 } 052 053 /** 054 * Starts the NamespaceStateManager. The boot strap of cache is done in the post master start hook 055 * of the NamespaceAuditor class. 056 * @throws IOException Signals that an I/O exception has occurred. 057 */ 058 public void start() throws IOException { 059 LOG.info("Namespace State Manager started."); 060 initialize(); 061 } 062 063 /** 064 * Gets an instance of NamespaceTableAndRegionInfo associated with namespace. 065 * @param name The name of the namespace 066 * @return An instance of NamespaceTableAndRegionInfo. 067 */ 068 public NamespaceTableAndRegionInfo getState(String name) { 069 return nsStateCache.get(name); 070 } 071 072 /** 073 * Check if adding a region violates namespace quota, if not update namespace cache. 074 * @return true, if region can be added to table. 075 * @throws IOException Signals that an I/O exception has occurred. 076 */ 077 synchronized boolean checkAndUpdateNamespaceRegionCount(TableName name, byte[] regionName, 078 int incr) throws IOException { 079 String namespace = name.getNamespaceAsString(); 080 NamespaceDescriptor nspdesc = getNamespaceDescriptor(namespace); 081 if (nspdesc != null) { 082 NamespaceTableAndRegionInfo currentStatus; 083 currentStatus = getState(namespace); 084 int regionCount = currentStatus.getRegionCount(); 085 long maxRegionCount = TableNamespaceManager.getMaxRegions(nspdesc); 086 if (incr > 0 && regionCount >= maxRegionCount) { 087 LOG.warn("The region " + Bytes.toStringBinary(regionName) 088 + " cannot be created. The region count will exceed quota on the namespace. " 089 + "This may be transient, please retry later if there are any ongoing split" 090 + " operations in the namespace."); 091 return false; 092 } 093 NamespaceTableAndRegionInfo nsInfo = nsStateCache.get(namespace); 094 if (nsInfo != null) { 095 nsInfo.incRegionCountForTable(name, incr); 096 } else { 097 LOG.warn("Namespace state found null for namespace : " + namespace); 098 } 099 } 100 return true; 101 } 102 103 /** 104 * Check and update region count for an existing table. To handle scenarios like restore snapshot 105 * @param name name of the table for region count needs to be checked and updated 106 * @param incr count of regions 107 * @throws QuotaExceededException if quota exceeds for the number of regions allowed in a 108 * namespace 109 * @throws IOException Signals that an I/O exception has occurred. 110 */ 111 synchronized void checkAndUpdateNamespaceRegionCount(TableName name, int incr) 112 throws IOException { 113 String namespace = name.getNamespaceAsString(); 114 NamespaceDescriptor nspdesc = getNamespaceDescriptor(namespace); 115 if (nspdesc != null) { 116 NamespaceTableAndRegionInfo currentStatus = getState(namespace); 117 int regionCountOfTable = currentStatus.getRegionCountOfTable(name); 118 if ( 119 (currentStatus.getRegionCount() - regionCountOfTable + incr) 120 > TableNamespaceManager.getMaxRegions(nspdesc) 121 ) { 122 throw new QuotaExceededException("The table " + name.getNameAsString() 123 + " region count cannot be updated as it would exceed maximum number " 124 + "of regions allowed in the namespace. The total number of regions permitted is " 125 + TableNamespaceManager.getMaxRegions(nspdesc)); 126 } 127 currentStatus.removeTable(name); 128 currentStatus.addTable(name, incr); 129 } 130 } 131 132 private NamespaceDescriptor getNamespaceDescriptor(String namespaceAsString) { 133 try { 134 return this.master.getClusterSchema().getNamespace(namespaceAsString); 135 } catch (IOException e) { 136 LOG.error("Error while fetching namespace descriptor for namespace : " + namespaceAsString); 137 return null; 138 } 139 } 140 141 synchronized void checkAndUpdateNamespaceTableCount(TableName table, int numRegions) 142 throws IOException { 143 String namespace = table.getNamespaceAsString(); 144 NamespaceDescriptor nspdesc = getNamespaceDescriptor(namespace); 145 if (nspdesc != null) { 146 NamespaceTableAndRegionInfo currentStatus; 147 currentStatus = getState(nspdesc.getName()); 148 if ((currentStatus.getTables().size()) >= TableNamespaceManager.getMaxTables(nspdesc)) { 149 throw new QuotaExceededException("The table " + table.getNameAsString() 150 + " cannot be created as it would exceed maximum number of tables allowed " 151 + " in the namespace. The total number of tables permitted is " 152 + TableNamespaceManager.getMaxTables(nspdesc)); 153 } 154 if ( 155 (currentStatus.getRegionCount() + numRegions) > TableNamespaceManager.getMaxRegions(nspdesc) 156 ) { 157 throw new QuotaExceededException( 158 "The table " + table.getNameAsString() + " is not allowed to have " + numRegions 159 + " regions. The total number of regions permitted is only " 160 + TableNamespaceManager.getMaxRegions(nspdesc) + ", while current region count is " 161 + currentStatus.getRegionCount() 162 + ". This may be transient, please retry later if there are any" 163 + " ongoing split operations in the namespace."); 164 } 165 } else { 166 throw new IOException( 167 "Namespace Descriptor found null for " + namespace + " This is unexpected."); 168 } 169 addTable(table, numRegions); 170 } 171 172 NamespaceTableAndRegionInfo addNamespace(String namespace) { 173 if (!nsStateCache.containsKey(namespace)) { 174 NamespaceTableAndRegionInfo a1 = new NamespaceTableAndRegionInfo(namespace); 175 nsStateCache.put(namespace, a1); 176 } 177 return nsStateCache.get(namespace); 178 } 179 180 /** 181 * Delete the namespace state. 182 * @param namespace the name of the namespace to delete 183 */ 184 void deleteNamespace(String namespace) { 185 this.nsStateCache.remove(namespace); 186 } 187 188 private void addTable(TableName tableName, int regionCount) throws IOException { 189 NamespaceTableAndRegionInfo info = nsStateCache.get(tableName.getNamespaceAsString()); 190 if (info != null) { 191 info.addTable(tableName, regionCount); 192 } else { 193 throw new IOException("Bad state : Namespace quota information not found for namespace : " 194 + tableName.getNamespaceAsString()); 195 } 196 } 197 198 synchronized void removeTable(TableName tableName) { 199 NamespaceTableAndRegionInfo info = nsStateCache.get(tableName.getNamespaceAsString()); 200 if (info != null) { 201 info.removeTable(tableName); 202 } 203 } 204 205 /** 206 * Initialize namespace state cache by scanning meta table. 207 */ 208 private void initialize() throws IOException { 209 List<NamespaceDescriptor> namespaces = this.master.getClusterSchema().getNamespaces(); 210 for (NamespaceDescriptor namespace : namespaces) { 211 addNamespace(namespace.getName()); 212 List<TableName> tables = this.master.listTableNamesByNamespace(namespace.getName()); 213 for (TableName table : tables) { 214 if (table.isSystemTable()) { 215 continue; 216 } 217 List<RegionInfo> regions = 218 MetaTableAccessor.getTableRegions(this.master.getConnection(), table, true); 219 addTable(table, regions.size()); 220 } 221 } 222 LOG.info("Finished updating state of " + nsStateCache.size() + " namespaces. "); 223 initialized = true; 224 } 225 226 boolean isInitialized() { 227 return initialized; 228 } 229 230 public synchronized void removeRegionFromTable(RegionInfo hri) throws IOException { 231 String namespace = hri.getTable().getNamespaceAsString(); 232 NamespaceTableAndRegionInfo nsInfo = nsStateCache.get(namespace); 233 if (nsInfo != null) { 234 nsInfo.decrementRegionCountForTable(hri.getTable(), 1); 235 } else { 236 throw new IOException("Namespace state found null for namespace : " + namespace); 237 } 238 } 239}