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.util; 019 020import java.io.IOException; 021import org.apache.hadoop.conf.Configuration; 022import org.apache.hadoop.hbase.CompoundConfiguration; 023import org.apache.hadoop.hbase.DoNotRetryIOException; 024import org.apache.hadoop.hbase.HConstants; 025import org.apache.hadoop.hbase.TableName; 026import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor; 027import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder; 028import org.apache.hadoop.hbase.client.TableDescriptor; 029import org.apache.hadoop.hbase.client.TableDescriptorBuilder; 030import org.apache.hadoop.hbase.regionserver.DefaultStoreEngine; 031import org.apache.hadoop.hbase.regionserver.HStore; 032import org.apache.hadoop.hbase.regionserver.RegionCoprocessorHost; 033import org.apache.hadoop.hbase.regionserver.RegionSplitPolicy; 034import org.apache.hadoop.hbase.regionserver.compactions.ExploringCompactionPolicy; 035import org.apache.hadoop.hbase.regionserver.compactions.FIFOCompactionPolicy; 036import org.apache.yetus.audience.InterfaceAudience; 037import org.slf4j.Logger; 038import org.slf4j.LoggerFactory; 039 040import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos; 041 042/** 043 * Only used for master to sanity check {@link org.apache.hadoop.hbase.client.TableDescriptor}. 044 */ 045@InterfaceAudience.Private 046public final class TableDescriptorChecker { 047 private static Logger LOG = LoggerFactory.getLogger(TableDescriptorChecker.class); 048 049 public static final String TABLE_SANITY_CHECKS = "hbase.table.sanity.checks"; 050 public static final boolean DEFAULT_TABLE_SANITY_CHECKS = true; 051 052 // should we check the compression codec type at master side, default true, HBASE-6370 053 public static final String MASTER_CHECK_COMPRESSION = "hbase.master.check.compression"; 054 public static final boolean DEFAULT_MASTER_CHECK_COMPRESSION = true; 055 056 // should we check encryption settings at master side, default true 057 public static final String MASTER_CHECK_ENCRYPTION = "hbase.master.check.encryption"; 058 public static final boolean DEFAULT_MASTER_CHECK_ENCRYPTION = true; 059 060 private TableDescriptorChecker() { 061 } 062 063 /** 064 * Checks whether the table conforms to some sane limits, and configured values (compression, etc) 065 * work. Throws an exception if something is wrong. 066 */ 067 public static void sanityCheck(final Configuration c, final TableDescriptor td) 068 throws IOException { 069 CompoundConfiguration conf = new CompoundConfiguration().add(c).addBytesMap(td.getValues()); 070 071 // Setting this to true logs the warning instead of throwing exception 072 boolean logWarn = false; 073 if (!conf.getBoolean(TABLE_SANITY_CHECKS, DEFAULT_TABLE_SANITY_CHECKS)) { 074 logWarn = true; 075 } 076 String tableVal = td.getValue(TABLE_SANITY_CHECKS); 077 if (tableVal != null && !Boolean.valueOf(tableVal)) { 078 logWarn = true; 079 } 080 081 // check max file size 082 long maxFileSizeLowerLimit = 2 * 1024 * 1024L; // 2M is the default lower limit 083 // if not set MAX_FILESIZE in TableDescriptor, and not set HREGION_MAX_FILESIZE in 084 // hbase-site.xml, use maxFileSizeLowerLimit instead to skip this check 085 long maxFileSize = td.getValue(TableDescriptorBuilder.MAX_FILESIZE) == null 086 ? conf.getLong(HConstants.HREGION_MAX_FILESIZE, maxFileSizeLowerLimit) 087 : Long.parseLong(td.getValue(TableDescriptorBuilder.MAX_FILESIZE)); 088 if (maxFileSize < conf.getLong("hbase.hregion.max.filesize.limit", maxFileSizeLowerLimit)) { 089 String message = "MAX_FILESIZE for table descriptor or " + "\"hbase.hregion.max.filesize\" (" 090 + maxFileSize + ") is too small, which might cause over splitting into unmanageable " 091 + "number of regions."; 092 warnOrThrowExceptionForFailure(logWarn, message, null); 093 } 094 095 // check flush size 096 long flushSizeLowerLimit = 1024 * 1024L; // 1M is the default lower limit 097 // if not set MEMSTORE_FLUSHSIZE in TableDescriptor, and not set HREGION_MEMSTORE_FLUSH_SIZE in 098 // hbase-site.xml, use flushSizeLowerLimit instead to skip this check 099 long flushSize = td.getValue(TableDescriptorBuilder.MEMSTORE_FLUSHSIZE) == null 100 ? conf.getLong(HConstants.HREGION_MEMSTORE_FLUSH_SIZE, flushSizeLowerLimit) 101 : Long.parseLong(td.getValue(TableDescriptorBuilder.MEMSTORE_FLUSHSIZE)); 102 if (flushSize < conf.getLong("hbase.hregion.memstore.flush.size.limit", flushSizeLowerLimit)) { 103 String message = 104 "MEMSTORE_FLUSHSIZE for table descriptor or " + "\"hbase.hregion.memstore.flush.size\" (" 105 + flushSize + ") is too small, which might cause" + " very frequent flushing."; 106 warnOrThrowExceptionForFailure(logWarn, message, null); 107 } 108 109 // check that coprocessors and other specified plugin classes can be loaded 110 try { 111 checkClassLoading(conf, td); 112 } catch (Exception ex) { 113 warnOrThrowExceptionForFailure(logWarn, ex.getMessage(), null); 114 } 115 116 if (conf.getBoolean(MASTER_CHECK_COMPRESSION, DEFAULT_MASTER_CHECK_COMPRESSION)) { 117 // check compression can be loaded 118 try { 119 checkCompression(td); 120 } catch (IOException e) { 121 warnOrThrowExceptionForFailure(logWarn, e.getMessage(), e); 122 } 123 } 124 125 if (conf.getBoolean(MASTER_CHECK_ENCRYPTION, DEFAULT_MASTER_CHECK_ENCRYPTION)) { 126 // check encryption can be loaded 127 try { 128 checkEncryption(conf, td); 129 } catch (IOException e) { 130 warnOrThrowExceptionForFailure(logWarn, e.getMessage(), e); 131 } 132 } 133 134 // Verify compaction policy 135 try { 136 checkCompactionPolicy(conf, td); 137 } catch (IOException e) { 138 warnOrThrowExceptionForFailure(false, e.getMessage(), e); 139 } 140 // check that we have at least 1 CF 141 if (td.getColumnFamilyCount() == 0) { 142 String message = "Table should have at least one column family."; 143 warnOrThrowExceptionForFailure(logWarn, message, null); 144 } 145 146 // check that we have minimum 1 region replicas 147 int regionReplicas = td.getRegionReplication(); 148 if (regionReplicas < 1) { 149 String message = "Table region replication should be at least one."; 150 warnOrThrowExceptionForFailure(logWarn, message, null); 151 } 152 153 // Meta table shouldn't be set as read only, otherwise it will impact region assignments 154 if (td.isReadOnly() && TableName.isMetaTableName(td.getTableName())) { 155 warnOrThrowExceptionForFailure(false, "Meta table can't be set as read only.", null); 156 } 157 158 for (ColumnFamilyDescriptor hcd : td.getColumnFamilies()) { 159 if (hcd.getTimeToLive() <= 0) { 160 String message = "TTL for column family " + hcd.getNameAsString() + " must be positive."; 161 warnOrThrowExceptionForFailure(logWarn, message, null); 162 } 163 164 // check blockSize 165 if (hcd.getBlocksize() < 1024 || hcd.getBlocksize() > 16 * 1024 * 1024) { 166 String message = "Block size for column family " + hcd.getNameAsString() 167 + " must be between 1K and 16MB."; 168 warnOrThrowExceptionForFailure(logWarn, message, null); 169 } 170 171 // check versions 172 if (hcd.getMinVersions() < 0) { 173 String message = 174 "Min versions for column family " + hcd.getNameAsString() + " must be positive."; 175 warnOrThrowExceptionForFailure(logWarn, message, null); 176 } 177 // max versions already being checked 178 179 // HBASE-13776 Setting illegal versions for ColumnFamilyDescriptor 180 // does not throw IllegalArgumentException 181 // check minVersions <= maxVerions 182 if (hcd.getMinVersions() > hcd.getMaxVersions()) { 183 String message = "Min versions for column family " + hcd.getNameAsString() 184 + " must be less than the Max versions."; 185 warnOrThrowExceptionForFailure(logWarn, message, null); 186 } 187 188 // check replication scope 189 checkReplicationScope(hcd); 190 // check bloom filter type 191 checkBloomFilterType(hcd); 192 193 // check data replication factor, it can be 0(default value) when user has not explicitly 194 // set the value, in this case we use default replication factor set in the file system. 195 if (hcd.getDFSReplication() < 0) { 196 String message = "HFile Replication for column family " + hcd.getNameAsString() 197 + " must be greater than zero."; 198 warnOrThrowExceptionForFailure(logWarn, message, null); 199 } 200 201 // check in-memory compaction 202 try { 203 hcd.getInMemoryCompaction(); 204 } catch (IllegalArgumentException e) { 205 warnOrThrowExceptionForFailure(logWarn, e.getMessage(), e); 206 } 207 } 208 } 209 210 private static void checkReplicationScope(final ColumnFamilyDescriptor cfd) throws IOException { 211 // check replication scope 212 WALProtos.ScopeType scop = WALProtos.ScopeType.valueOf(cfd.getScope()); 213 if (scop == null) { 214 String message = "Replication scope for column family " + cfd.getNameAsString() + " is " 215 + cfd.getScope() + " which is invalid."; 216 217 LOG.error(message); 218 throw new DoNotRetryIOException(message); 219 } 220 } 221 222 private static void checkCompactionPolicy(Configuration conf, TableDescriptor td) 223 throws IOException { 224 // FIFO compaction has some requirements 225 // Actually FCP ignores periodic major compactions 226 String className = td.getValue(DefaultStoreEngine.DEFAULT_COMPACTION_POLICY_CLASS_KEY); 227 if (className == null) { 228 className = conf.get(DefaultStoreEngine.DEFAULT_COMPACTION_POLICY_CLASS_KEY, 229 ExploringCompactionPolicy.class.getName()); 230 } 231 232 int blockingFileCount = HStore.DEFAULT_BLOCKING_STOREFILE_COUNT; 233 String sv = td.getValue(HStore.BLOCKING_STOREFILES_KEY); 234 if (sv != null) { 235 blockingFileCount = Integer.parseInt(sv); 236 } else { 237 blockingFileCount = conf.getInt(HStore.BLOCKING_STOREFILES_KEY, blockingFileCount); 238 } 239 240 for (ColumnFamilyDescriptor hcd : td.getColumnFamilies()) { 241 String compactionPolicy = 242 hcd.getConfigurationValue(DefaultStoreEngine.DEFAULT_COMPACTION_POLICY_CLASS_KEY); 243 if (compactionPolicy == null) { 244 compactionPolicy = className; 245 } 246 if (!compactionPolicy.equals(FIFOCompactionPolicy.class.getName())) { 247 continue; 248 } 249 // FIFOCompaction 250 String message = null; 251 252 // 1. Check TTL 253 if (hcd.getTimeToLive() == ColumnFamilyDescriptorBuilder.DEFAULT_TTL) { 254 message = "Default TTL is not supported for FIFO compaction"; 255 throw new IOException(message); 256 } 257 258 // 2. Check min versions 259 if (hcd.getMinVersions() > 0) { 260 message = "MIN_VERSION > 0 is not supported for FIFO compaction"; 261 throw new IOException(message); 262 } 263 264 // 3. blocking file count 265 sv = hcd.getConfigurationValue(HStore.BLOCKING_STOREFILES_KEY); 266 if (sv != null) { 267 blockingFileCount = Integer.parseInt(sv); 268 } 269 if (blockingFileCount < 1000) { 270 message = 271 "Blocking file count '" + HStore.BLOCKING_STOREFILES_KEY + "' " + blockingFileCount 272 + " is below recommended minimum of 1000 for column family " + hcd.getNameAsString(); 273 throw new IOException(message); 274 } 275 } 276 } 277 278 private static void checkBloomFilterType(ColumnFamilyDescriptor cfd) throws IOException { 279 Configuration conf = new CompoundConfiguration().addStringMap(cfd.getConfiguration()); 280 try { 281 BloomFilterUtil.getBloomFilterParam(cfd.getBloomFilterType(), conf); 282 } catch (IllegalArgumentException e) { 283 throw new DoNotRetryIOException("Failed to get bloom filter param", e); 284 } 285 } 286 287 public static void checkCompression(final TableDescriptor td) throws IOException { 288 for (ColumnFamilyDescriptor cfd : td.getColumnFamilies()) { 289 CompressionTest.testCompression(cfd.getCompressionType()); 290 CompressionTest.testCompression(cfd.getCompactionCompressionType()); 291 CompressionTest.testCompression(cfd.getMajorCompactionCompressionType()); 292 CompressionTest.testCompression(cfd.getMinorCompactionCompressionType()); 293 } 294 } 295 296 public static void checkEncryption(final Configuration conf, final TableDescriptor td) 297 throws IOException { 298 for (ColumnFamilyDescriptor cfd : td.getColumnFamilies()) { 299 EncryptionTest.testEncryption(conf, cfd.getEncryptionType(), cfd.getEncryptionKey()); 300 } 301 } 302 303 public static void checkClassLoading(final Configuration conf, final TableDescriptor td) 304 throws IOException { 305 RegionSplitPolicy.getSplitPolicyClass(td, conf); 306 RegionCoprocessorHost.testTableCoprocessorAttrs(conf, td); 307 } 308 309 // HBASE-13350 - Helper method to log warning on sanity check failures if checks disabled. 310 private static void warnOrThrowExceptionForFailure(boolean logWarn, String message, 311 Exception cause) throws IOException { 312 if (!logWarn) { 313 throw new DoNotRetryIOException(message + " Set " + TABLE_SANITY_CHECKS 314 + " to false at conf or table descriptor if you want to bypass sanity checks", cause); 315 } 316 LOG.warn(message); 317 } 318}