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.regionserver; 019 020import java.io.FileNotFoundException; 021import java.io.IOException; 022import java.util.ArrayList; 023import java.util.Date; 024import java.util.List; 025import java.util.Map; 026import java.util.NavigableSet; 027import java.util.UUID; 028import java.util.concurrent.ConcurrentHashMap; 029import java.util.concurrent.atomic.AtomicLong; 030 031import org.apache.hadoop.conf.Configuration; 032import org.apache.hadoop.fs.FileSystem; 033import org.apache.hadoop.fs.Path; 034import org.apache.hadoop.hbase.ArrayBackedTag; 035import org.apache.hadoop.hbase.Cell; 036import org.apache.hadoop.hbase.CellBuilderType; 037import org.apache.hadoop.hbase.CellComparator; 038import org.apache.hadoop.hbase.DoNotRetryIOException; 039import org.apache.hadoop.hbase.ExtendedCellBuilderFactory; 040import org.apache.hadoop.hbase.HConstants; 041import org.apache.hadoop.hbase.TableName; 042import org.apache.hadoop.hbase.Tag; 043import org.apache.hadoop.hbase.TagType; 044import org.apache.hadoop.hbase.TagUtil; 045import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor; 046import org.apache.hadoop.hbase.client.Scan; 047import org.apache.hadoop.hbase.filter.Filter; 048import org.apache.hadoop.hbase.filter.FilterList; 049import org.apache.hadoop.hbase.io.compress.Compression; 050import org.apache.hadoop.hbase.io.hfile.CorruptHFileException; 051import org.apache.hadoop.hbase.mob.MobCell; 052import org.apache.hadoop.hbase.mob.MobConstants; 053import org.apache.hadoop.hbase.mob.MobFile; 054import org.apache.hadoop.hbase.mob.MobFileCache; 055import org.apache.hadoop.hbase.mob.MobFileName; 056import org.apache.hadoop.hbase.mob.MobStoreEngine; 057import org.apache.hadoop.hbase.mob.MobUtils; 058import org.apache.hadoop.hbase.util.Bytes; 059import org.apache.hadoop.hbase.util.HFileArchiveUtil; 060import org.apache.hadoop.hbase.util.IdLock; 061import org.apache.yetus.audience.InterfaceAudience; 062import org.slf4j.Logger; 063import org.slf4j.LoggerFactory; 064 065/** 066 * The store implementation to save MOBs (medium objects), it extends the HStore. 067 * When a descriptor of a column family has the value "IS_MOB", it means this column family 068 * is a mob one. When a HRegion instantiate a store for this column family, the HMobStore is 069 * created. 070 * HMobStore is almost the same with the HStore except using different types of scanners. 071 * In the method of getScanner, the MobStoreScanner and MobReversedStoreScanner are returned. 072 * In these scanners, a additional seeks in the mob files should be performed after the seek 073 * to HBase is done. 074 * The store implements how we save MOBs by extending HStore. When a descriptor 075 * of a column family has the value "IS_MOB", it means this column family is a mob one. When a 076 * HRegion instantiate a store for this column family, the HMobStore is created. HMobStore is 077 * almost the same with the HStore except using different types of scanners. In the method of 078 * getScanner, the MobStoreScanner and MobReversedStoreScanner are returned. In these scanners, a 079 * additional seeks in the mob files should be performed after the seek in HBase is done. 080 */ 081@InterfaceAudience.Private 082public class HMobStore extends HStore { 083 private static final Logger LOG = LoggerFactory.getLogger(HMobStore.class); 084 private MobFileCache mobFileCache; 085 private Path homePath; 086 private Path mobFamilyPath; 087 private AtomicLong cellsCountCompactedToMob = new AtomicLong(); 088 private AtomicLong cellsCountCompactedFromMob = new AtomicLong(); 089 private AtomicLong cellsSizeCompactedToMob = new AtomicLong(); 090 private AtomicLong cellsSizeCompactedFromMob = new AtomicLong(); 091 private AtomicLong mobFlushCount = new AtomicLong(); 092 private AtomicLong mobFlushedCellsCount = new AtomicLong(); 093 private AtomicLong mobFlushedCellsSize = new AtomicLong(); 094 private AtomicLong mobScanCellsCount = new AtomicLong(); 095 private AtomicLong mobScanCellsSize = new AtomicLong(); 096 private ColumnFamilyDescriptor family; 097 private Map<String, List<Path>> map = new ConcurrentHashMap<>(); 098 private final IdLock keyLock = new IdLock(); 099 // When we add a MOB reference cell to the HFile, we will add 2 tags along with it 100 // 1. A ref tag with type TagType.MOB_REFERENCE_TAG_TYPE. This just denote this this cell is not 101 // original one but a ref to another MOB Cell. 102 // 2. Table name tag. It's very useful in cloning the snapshot. When reading from the cloning 103 // table, we need to find the original mob files by this table name. For details please see 104 // cloning snapshot for mob files. 105 private final byte[] refCellTags; 106 107 public HMobStore(final HRegion region, final ColumnFamilyDescriptor family, 108 final Configuration confParam, boolean warmup) throws IOException { 109 super(region, family, confParam, warmup); 110 this.family = family; 111 this.mobFileCache = region.getMobFileCache(); 112 this.homePath = MobUtils.getMobHome(conf); 113 this.mobFamilyPath = MobUtils.getMobFamilyPath(conf, this.getTableName(), 114 family.getNameAsString()); 115 List<Path> locations = new ArrayList<>(2); 116 locations.add(mobFamilyPath); 117 TableName tn = region.getTableDescriptor().getTableName(); 118 locations.add(HFileArchiveUtil.getStoreArchivePath(conf, tn, MobUtils.getMobRegionInfo(tn) 119 .getEncodedName(), family.getNameAsString())); 120 map.put(Bytes.toString(tn.getName()), locations); 121 List<Tag> tags = new ArrayList<>(2); 122 tags.add(MobConstants.MOB_REF_TAG); 123 Tag tableNameTag = new ArrayBackedTag(TagType.MOB_TABLE_NAME_TAG_TYPE, 124 getTableName().getName()); 125 tags.add(tableNameTag); 126 this.refCellTags = TagUtil.fromList(tags); 127 } 128 129 /** 130 * Gets current config. 131 */ 132 public Configuration getConfiguration() { 133 return this.conf; 134 } 135 136 /** 137 * Gets the MobStoreScanner or MobReversedStoreScanner. In these scanners, a additional seeks in 138 * the mob files should be performed after the seek in HBase is done. 139 */ 140 @Override 141 protected KeyValueScanner createScanner(Scan scan, ScanInfo scanInfo, 142 NavigableSet<byte[]> targetCols, long readPt) throws IOException { 143 if (MobUtils.isRefOnlyScan(scan)) { 144 Filter refOnlyFilter = new MobReferenceOnlyFilter(); 145 Filter filter = scan.getFilter(); 146 if (filter != null) { 147 scan.setFilter(new FilterList(filter, refOnlyFilter)); 148 } else { 149 scan.setFilter(refOnlyFilter); 150 } 151 } 152 return scan.isReversed() ? new ReversedMobStoreScanner(this, scanInfo, scan, targetCols, readPt) 153 : new MobStoreScanner(this, scanInfo, scan, targetCols, readPt); 154 } 155 156 /** 157 * Creates the mob store engine. 158 */ 159 @Override 160 protected StoreEngine<?, ?, ?, ?> createStoreEngine(HStore store, Configuration conf, 161 CellComparator cellComparator) throws IOException { 162 MobStoreEngine engine = new MobStoreEngine(); 163 engine.createComponents(conf, store, cellComparator); 164 return engine; 165 } 166 167 /** 168 * Gets the temp directory. 169 * @return The temp directory. 170 */ 171 private Path getTempDir() { 172 return new Path(homePath, MobConstants.TEMP_DIR_NAME); 173 } 174 175 /** 176 * Creates the writer for the mob file in temp directory. 177 * @param date The latest date of written cells. 178 * @param maxKeyCount The key count. 179 * @param compression The compression algorithm. 180 * @param startKey The start key. 181 * @param isCompaction If the writer is used in compaction. 182 * @return The writer for the mob file. 183 * @throws IOException 184 */ 185 public StoreFileWriter createWriterInTmp(Date date, long maxKeyCount, 186 Compression.Algorithm compression, byte[] startKey, 187 boolean isCompaction) throws IOException { 188 if (startKey == null) { 189 startKey = HConstants.EMPTY_START_ROW; 190 } 191 Path path = getTempDir(); 192 return createWriterInTmp(MobUtils.formatDate(date), path, maxKeyCount, compression, startKey, 193 isCompaction); 194 } 195 196 /** 197 * Creates the writer for the del file in temp directory. 198 * The del file keeps tracking the delete markers. Its name has a suffix _del, 199 * the format is [0-9a-f]+(_del)?. 200 * @param date The latest date of written cells. 201 * @param maxKeyCount The key count. 202 * @param compression The compression algorithm. 203 * @param startKey The start key. 204 * @return The writer for the del file. 205 * @throws IOException 206 */ 207 public StoreFileWriter createDelFileWriterInTmp(Date date, long maxKeyCount, 208 Compression.Algorithm compression, byte[] startKey) throws IOException { 209 if (startKey == null) { 210 startKey = HConstants.EMPTY_START_ROW; 211 } 212 Path path = getTempDir(); 213 String suffix = UUID 214 .randomUUID().toString().replaceAll("-", "") + "_del"; 215 MobFileName mobFileName = MobFileName.create(startKey, MobUtils.formatDate(date), suffix); 216 return createWriterInTmp(mobFileName, path, maxKeyCount, compression, true); 217 } 218 219 /** 220 * Creates the writer for the mob file in temp directory. 221 * @param date The date string, its format is yyyymmmdd. 222 * @param basePath The basic path for a temp directory. 223 * @param maxKeyCount The key count. 224 * @param compression The compression algorithm. 225 * @param startKey The start key. 226 * @param isCompaction If the writer is used in compaction. 227 * @return The writer for the mob file. 228 * @throws IOException 229 */ 230 public StoreFileWriter createWriterInTmp(String date, Path basePath, long maxKeyCount, 231 Compression.Algorithm compression, byte[] startKey, 232 boolean isCompaction) throws IOException { 233 MobFileName mobFileName = MobFileName.create(startKey, date, UUID.randomUUID() 234 .toString().replaceAll("-", "")); 235 return createWriterInTmp(mobFileName, basePath, maxKeyCount, compression, isCompaction); 236 } 237 238 /** 239 * Creates the writer for the mob file in temp directory. 240 * @param mobFileName The mob file name. 241 * @param basePath The basic path for a temp directory. 242 * @param maxKeyCount The key count. 243 * @param compression The compression algorithm. 244 * @param isCompaction If the writer is used in compaction. 245 * @return The writer for the mob file. 246 * @throws IOException 247 */ 248 public StoreFileWriter createWriterInTmp(MobFileName mobFileName, Path basePath, 249 long maxKeyCount, Compression.Algorithm compression, 250 boolean isCompaction) throws IOException { 251 return MobUtils.createWriter(conf, region.getFilesystem(), family, 252 new Path(basePath, mobFileName.getFileName()), maxKeyCount, compression, cacheConf, 253 cryptoContext, checksumType, bytesPerChecksum, blocksize, BloomType.NONE, isCompaction); 254 } 255 256 /** 257 * Commits the mob file. 258 * @param sourceFile The source file. 259 * @param targetPath The directory path where the source file is renamed to. 260 * @throws IOException 261 */ 262 public void commitFile(final Path sourceFile, Path targetPath) throws IOException { 263 if (sourceFile == null) { 264 return; 265 } 266 Path dstPath = new Path(targetPath, sourceFile.getName()); 267 validateMobFile(sourceFile); 268 String msg = "Renaming flushed file from " + sourceFile + " to " + dstPath; 269 LOG.info(msg); 270 Path parent = dstPath.getParent(); 271 if (!region.getFilesystem().exists(parent)) { 272 region.getFilesystem().mkdirs(parent); 273 } 274 if (!region.getFilesystem().rename(sourceFile, dstPath)) { 275 throw new IOException("Failed rename of " + sourceFile + " to " + dstPath); 276 } 277 } 278 279 /** 280 * Validates a mob file by opening and closing it. 281 * 282 * @param path the path to the mob file 283 */ 284 private void validateMobFile(Path path) throws IOException { 285 HStoreFile storeFile = null; 286 try { 287 storeFile = new HStoreFile(region.getFilesystem(), path, conf, this.cacheConf, 288 BloomType.NONE, isPrimaryReplicaStore()); 289 storeFile.initReader(); 290 } catch (IOException e) { 291 LOG.error("Fail to open mob file[" + path + "], keep it in temp directory.", e); 292 throw e; 293 } finally { 294 if (storeFile != null) { 295 storeFile.closeStoreFile(false); 296 } 297 } 298 } 299 300 /** 301 * Reads the cell from the mob file, and the read point does not count. This is used for 302 * DefaultMobStoreCompactor where we can read empty value for the missing cell. 303 * @param reference The cell found in the HBase, its value is a path to a mob file. 304 * @param cacheBlocks Whether the scanner should cache blocks. 305 * @return The cell found in the mob file. 306 * @throws IOException 307 */ 308 public MobCell resolve(Cell reference, boolean cacheBlocks) throws IOException { 309 return resolve(reference, cacheBlocks, -1, true); 310 } 311 312 /** 313 * Reads the cell from the mob file. 314 * @param reference The cell found in the HBase, its value is a path to a mob file. 315 * @param cacheBlocks Whether the scanner should cache blocks. 316 * @param readPt the read point. 317 * @param readEmptyValueOnMobCellMiss Whether return null value when the mob file is missing or 318 * corrupt. 319 * @return The cell found in the mob file. 320 * @throws IOException 321 */ 322 public MobCell resolve(Cell reference, boolean cacheBlocks, long readPt, 323 boolean readEmptyValueOnMobCellMiss) throws IOException { 324 MobCell mobCell = null; 325 if (MobUtils.hasValidMobRefCellValue(reference)) { 326 String fileName = MobUtils.getMobFileName(reference); 327 Tag tableNameTag = MobUtils.getTableNameTag(reference); 328 if (tableNameTag != null) { 329 String tableNameString = Tag.getValueAsString(tableNameTag); 330 List<Path> locations = map.get(tableNameString); 331 if (locations == null) { 332 IdLock.Entry lockEntry = keyLock.getLockEntry(tableNameString.hashCode()); 333 try { 334 locations = map.get(tableNameString); 335 if (locations == null) { 336 locations = new ArrayList<>(2); 337 TableName tn = TableName.valueOf(tableNameString); 338 locations.add(MobUtils.getMobFamilyPath(conf, tn, family.getNameAsString())); 339 locations.add(HFileArchiveUtil.getStoreArchivePath(conf, tn, 340 MobUtils.getMobRegionInfo(tn).getEncodedName(), family.getNameAsString())); 341 map.put(tableNameString, locations); 342 } 343 } finally { 344 keyLock.releaseLockEntry(lockEntry); 345 } 346 } 347 mobCell = readCell(locations, fileName, reference, cacheBlocks, readPt, 348 readEmptyValueOnMobCellMiss); 349 } 350 } 351 if (mobCell == null) { 352 LOG.warn("The Cell result is null, assemble a new Cell with the same row,family," 353 + "qualifier,timestamp,type and tags but with an empty value to return."); 354 Cell cell = ExtendedCellBuilderFactory.create(CellBuilderType.DEEP_COPY) 355 .setRow(reference.getRowArray(), reference.getRowOffset(), reference.getRowLength()) 356 .setFamily(reference.getFamilyArray(), reference.getFamilyOffset(), 357 reference.getFamilyLength()) 358 .setQualifier(reference.getQualifierArray(), reference.getQualifierOffset(), 359 reference.getQualifierLength()) 360 .setTimestamp(reference.getTimestamp()).setType(reference.getTypeByte()) 361 .setValue(HConstants.EMPTY_BYTE_ARRAY) 362 .setTags(reference.getTagsArray(), reference.getTagsOffset(), reference.getTagsLength()) 363 .build(); 364 mobCell = new MobCell(cell); 365 } 366 return mobCell; 367 } 368 369 /** 370 * Reads the cell from a mob file. 371 * The mob file might be located in different directories. 372 * 1. The working directory. 373 * 2. The archive directory. 374 * Reads the cell from the files located in both of the above directories. 375 * @param locations The possible locations where the mob files are saved. 376 * @param fileName The file to be read. 377 * @param search The cell to be searched. 378 * @param cacheMobBlocks Whether the scanner should cache blocks. 379 * @param readPt the read point. 380 * @param readEmptyValueOnMobCellMiss Whether return null value when the mob file is 381 * missing or corrupt. 382 * @return The found cell. Null if there's no such a cell. 383 * @throws IOException 384 */ 385 private MobCell readCell(List<Path> locations, String fileName, Cell search, 386 boolean cacheMobBlocks, long readPt, boolean readEmptyValueOnMobCellMiss) throws IOException { 387 FileSystem fs = getFileSystem(); 388 Throwable throwable = null; 389 for (Path location : locations) { 390 MobFile file = null; 391 Path path = new Path(location, fileName); 392 try { 393 file = mobFileCache.openFile(fs, path, cacheConf); 394 return readPt != -1 ? file.readCell(search, cacheMobBlocks, readPt) 395 : file.readCell(search, cacheMobBlocks); 396 } catch (IOException e) { 397 mobFileCache.evictFile(fileName); 398 throwable = e; 399 if ((e instanceof FileNotFoundException) || 400 (e.getCause() instanceof FileNotFoundException)) { 401 LOG.debug("Fail to read the cell, the mob file " + path + " doesn't exist", e); 402 } else if (e instanceof CorruptHFileException) { 403 LOG.error("The mob file " + path + " is corrupt", e); 404 break; 405 } else { 406 throw e; 407 } 408 } catch (NullPointerException e) { // HDFS 1.x - DFSInputStream.getBlockAt() 409 mobFileCache.evictFile(fileName); 410 LOG.debug("Fail to read the cell", e); 411 throwable = e; 412 } catch (AssertionError e) { // assert in HDFS 1.x - DFSInputStream.getBlockAt() 413 mobFileCache.evictFile(fileName); 414 LOG.debug("Fail to read the cell", e); 415 throwable = e; 416 } finally { 417 if (file != null) { 418 mobFileCache.closeFile(file); 419 } 420 } 421 } 422 LOG.error("The mob file " + fileName + " could not be found in the locations " + locations 423 + " or it is corrupt"); 424 if (readEmptyValueOnMobCellMiss) { 425 return null; 426 } else if ((throwable instanceof FileNotFoundException) 427 || (throwable.getCause() instanceof FileNotFoundException)) { 428 // The region is re-opened when FileNotFoundException is thrown. 429 // This is not necessary when MOB files cannot be found, because the store files 430 // in a region only contain the references to MOB files and a re-open on a region 431 // doesn't help fix the lost MOB files. 432 throw new DoNotRetryIOException(throwable); 433 } else if (throwable instanceof IOException) { 434 throw (IOException) throwable; 435 } else { 436 throw new IOException(throwable); 437 } 438 } 439 440 /** 441 * Gets the mob file path. 442 * @return The mob file path. 443 */ 444 public Path getPath() { 445 return mobFamilyPath; 446 } 447 448 public void updateCellsCountCompactedToMob(long count) { 449 cellsCountCompactedToMob.addAndGet(count); 450 } 451 452 public long getCellsCountCompactedToMob() { 453 return cellsCountCompactedToMob.get(); 454 } 455 456 public void updateCellsCountCompactedFromMob(long count) { 457 cellsCountCompactedFromMob.addAndGet(count); 458 } 459 460 public long getCellsCountCompactedFromMob() { 461 return cellsCountCompactedFromMob.get(); 462 } 463 464 public void updateCellsSizeCompactedToMob(long size) { 465 cellsSizeCompactedToMob.addAndGet(size); 466 } 467 468 public long getCellsSizeCompactedToMob() { 469 return cellsSizeCompactedToMob.get(); 470 } 471 472 public void updateCellsSizeCompactedFromMob(long size) { 473 cellsSizeCompactedFromMob.addAndGet(size); 474 } 475 476 public long getCellsSizeCompactedFromMob() { 477 return cellsSizeCompactedFromMob.get(); 478 } 479 480 public void updateMobFlushCount() { 481 mobFlushCount.incrementAndGet(); 482 } 483 484 public long getMobFlushCount() { 485 return mobFlushCount.get(); 486 } 487 488 public void updateMobFlushedCellsCount(long count) { 489 mobFlushedCellsCount.addAndGet(count); 490 } 491 492 public long getMobFlushedCellsCount() { 493 return mobFlushedCellsCount.get(); 494 } 495 496 public void updateMobFlushedCellsSize(long size) { 497 mobFlushedCellsSize.addAndGet(size); 498 } 499 500 public long getMobFlushedCellsSize() { 501 return mobFlushedCellsSize.get(); 502 } 503 504 public void updateMobScanCellsCount(long count) { 505 mobScanCellsCount.addAndGet(count); 506 } 507 508 public long getMobScanCellsCount() { 509 return mobScanCellsCount.get(); 510 } 511 512 public void updateMobScanCellsSize(long size) { 513 mobScanCellsSize.addAndGet(size); 514 } 515 516 public long getMobScanCellsSize() { 517 return mobScanCellsSize.get(); 518 } 519 520 public byte[] getRefCellTags() { 521 return this.refCellTags; 522 } 523}