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