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.snapshot; 019 020import java.io.FileNotFoundException; 021import java.io.IOException; 022import java.io.InterruptedIOException; 023import java.util.ArrayList; 024import java.util.Collection; 025import java.util.HashMap; 026import java.util.List; 027import java.util.Map; 028import java.util.concurrent.ExecutionException; 029import java.util.concurrent.ExecutorCompletionService; 030import java.util.concurrent.ThreadPoolExecutor; 031import java.util.concurrent.TimeUnit; 032import org.apache.hadoop.conf.Configuration; 033import org.apache.hadoop.fs.FSDataInputStream; 034import org.apache.hadoop.fs.FSDataOutputStream; 035import org.apache.hadoop.fs.FileStatus; 036import org.apache.hadoop.fs.FileSystem; 037import org.apache.hadoop.fs.Path; 038import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor; 039import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder; 040import org.apache.hadoop.hbase.client.RegionInfo; 041import org.apache.hadoop.hbase.client.TableDescriptor; 042import org.apache.hadoop.hbase.errorhandling.ForeignExceptionSnare; 043import org.apache.hadoop.hbase.mob.MobUtils; 044import org.apache.hadoop.hbase.monitoring.MonitoredTask; 045import org.apache.hadoop.hbase.regionserver.HRegion; 046import org.apache.hadoop.hbase.regionserver.HRegionFileSystem; 047import org.apache.hadoop.hbase.regionserver.HStore; 048import org.apache.hadoop.hbase.regionserver.HStoreFile; 049import org.apache.hadoop.hbase.regionserver.StoreFileInfo; 050import org.apache.hadoop.hbase.regionserver.storefiletracker.StoreFileTracker; 051import org.apache.hadoop.hbase.regionserver.storefiletracker.StoreFileTrackerFactory; 052import org.apache.hadoop.hbase.util.CommonFSUtils; 053import org.apache.hadoop.hbase.util.FSTableDescriptors; 054import org.apache.hadoop.hbase.util.Threads; 055import org.apache.yetus.audience.InterfaceAudience; 056import org.slf4j.Logger; 057import org.slf4j.LoggerFactory; 058 059import org.apache.hbase.thirdparty.com.google.common.util.concurrent.ThreadFactoryBuilder; 060import org.apache.hbase.thirdparty.com.google.protobuf.CodedInputStream; 061import org.apache.hbase.thirdparty.com.google.protobuf.InvalidProtocolBufferException; 062 063import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 064import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDataManifest; 065import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription; 066import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest; 067 068/** 069 * Utility class to help read/write the Snapshot Manifest. The snapshot format is transparent for 070 * the users of this class, once the snapshot is written, it will never be modified. On open() the 071 * snapshot will be loaded to the current in-memory format. 072 */ 073@InterfaceAudience.Private 074public final class SnapshotManifest { 075 private static final Logger LOG = LoggerFactory.getLogger(SnapshotManifest.class); 076 077 public static final String SNAPSHOT_MANIFEST_SIZE_LIMIT_CONF_KEY = "snapshot.manifest.size.limit"; 078 079 public static final String DATA_MANIFEST_NAME = "data.manifest"; 080 081 private List<SnapshotRegionManifest> regionManifests; 082 private SnapshotDescription desc; 083 private TableDescriptor htd; 084 085 private final ForeignExceptionSnare monitor; 086 private final Configuration conf; 087 private final Path workingDir; 088 private final FileSystem rootFs; 089 private final FileSystem workingDirFs; 090 private int manifestSizeLimit; 091 private final MonitoredTask statusTask; 092 093 /** 094 * @param conf configuration file for HBase setup 095 * @param rootFs root filesystem containing HFiles 096 * @param workingDir file path of where the manifest should be located 097 * @param desc description of snapshot being taken 098 * @param monitor monitor of foreign exceptions 099 * @throws IOException if the working directory file system cannot be determined from the config 100 * file 101 */ 102 private SnapshotManifest(final Configuration conf, final FileSystem rootFs, final Path workingDir, 103 final SnapshotDescription desc, final ForeignExceptionSnare monitor, 104 final MonitoredTask statusTask) throws IOException { 105 this.monitor = monitor; 106 this.desc = desc; 107 this.workingDir = workingDir; 108 this.conf = conf; 109 this.rootFs = rootFs; 110 this.statusTask = statusTask; 111 this.workingDirFs = this.workingDir.getFileSystem(this.conf); 112 this.manifestSizeLimit = conf.getInt(SNAPSHOT_MANIFEST_SIZE_LIMIT_CONF_KEY, 64 * 1024 * 1024); 113 } 114 115 /** 116 * Return a SnapshotManifest instance, used for writing a snapshot. There are two usage pattern: - 117 * The Master will create a manifest, add the descriptor, offline regions and consolidate the 118 * snapshot by writing all the pending stuff on-disk. manifest = SnapshotManifest.create(...) 119 * manifest.addRegion(tableDir, hri) manifest.consolidate() - The RegionServer will create a 120 * single region manifest manifest = SnapshotManifest.create(...) manifest.addRegion(region) 121 */ 122 public static SnapshotManifest create(final Configuration conf, final FileSystem fs, 123 final Path workingDir, final SnapshotDescription desc, final ForeignExceptionSnare monitor) 124 throws IOException { 125 return create(conf, fs, workingDir, desc, monitor, null); 126 127 } 128 129 public static SnapshotManifest create(final Configuration conf, final FileSystem fs, 130 final Path workingDir, final SnapshotDescription desc, final ForeignExceptionSnare monitor, 131 final MonitoredTask statusTask) throws IOException { 132 return new SnapshotManifest(conf, fs, workingDir, desc, monitor, statusTask); 133 134 } 135 136 /** 137 * Return a SnapshotManifest instance with the information already loaded in-memory. 138 * SnapshotManifest manifest = SnapshotManifest.open(...) TableDescriptor htd = 139 * manifest.getTableDescriptor() for (SnapshotRegionManifest regionManifest: 140 * manifest.getRegionManifests()) hri = regionManifest.getRegionInfo() for 141 * (regionManifest.getFamilyFiles()) ... 142 */ 143 public static SnapshotManifest open(final Configuration conf, final FileSystem fs, 144 final Path workingDir, final SnapshotDescription desc) throws IOException { 145 SnapshotManifest manifest = new SnapshotManifest(conf, fs, workingDir, desc, null, null); 146 manifest.load(); 147 return manifest; 148 } 149 150 /** 151 * Add the table descriptor to the snapshot manifest 152 */ 153 public void addTableDescriptor(final TableDescriptor htd) throws IOException { 154 this.htd = htd; 155 } 156 157 interface RegionVisitor<TRegion, TFamily> { 158 TRegion regionOpen(final RegionInfo regionInfo) throws IOException; 159 160 void regionClose(final TRegion region) throws IOException; 161 162 TFamily familyOpen(final TRegion region, final byte[] familyName) throws IOException; 163 164 void familyClose(final TRegion region, final TFamily family) throws IOException; 165 166 void storeFile(final TRegion region, final TFamily family, final StoreFileInfo storeFile) 167 throws IOException; 168 } 169 170 private RegionVisitor createRegionVisitor(final SnapshotDescription desc) throws IOException { 171 switch (getSnapshotFormat(desc)) { 172 case SnapshotManifestV1.DESCRIPTOR_VERSION: 173 return new SnapshotManifestV1.ManifestBuilder(conf, rootFs, workingDir); 174 case SnapshotManifestV2.DESCRIPTOR_VERSION: 175 return new SnapshotManifestV2.ManifestBuilder(conf, rootFs, workingDir); 176 default: 177 throw new CorruptedSnapshotException("Invalid Snapshot version: " + desc.getVersion(), 178 ProtobufUtil.createSnapshotDesc(desc)); 179 } 180 } 181 182 public void addMobRegion(RegionInfo regionInfo) throws IOException { 183 // Get the ManifestBuilder/RegionVisitor 184 RegionVisitor visitor = createRegionVisitor(desc); 185 186 // Visit the region and add it to the manifest 187 addMobRegion(regionInfo, visitor); 188 } 189 190 protected void addMobRegion(RegionInfo regionInfo, RegionVisitor visitor) throws IOException { 191 // 1. dump region meta info into the snapshot directory 192 final String snapshotName = desc.getName(); 193 LOG.debug("Storing mob region '" + regionInfo + "' region-info for snapshot=" + snapshotName); 194 Object regionData = visitor.regionOpen(regionInfo); 195 monitor.rethrowException(); 196 197 // 2. iterate through all the stores in the region 198 LOG.debug("Creating references for mob files"); 199 200 Path mobRegionPath = MobUtils.getMobRegionPath(conf, regionInfo.getTable()); 201 for (ColumnFamilyDescriptor hcd : htd.getColumnFamilies()) { 202 // 2.1. build the snapshot reference for the store if it's a mob store 203 if (!hcd.isMobEnabled()) { 204 continue; 205 } 206 Object familyData = visitor.familyOpen(regionData, hcd.getName()); 207 monitor.rethrowException(); 208 209 Path storePath = MobUtils.getMobFamilyPath(mobRegionPath, hcd.getNameAsString()); 210 List<StoreFileInfo> storeFiles = getStoreFiles(storePath); 211 if (storeFiles == null) { 212 if (LOG.isDebugEnabled()) { 213 LOG.debug("No mob files under family: " + hcd.getNameAsString()); 214 } 215 continue; 216 } 217 218 addReferenceFiles(visitor, regionData, familyData, storeFiles, true); 219 220 visitor.familyClose(regionData, familyData); 221 } 222 visitor.regionClose(regionData); 223 } 224 225 /** 226 * Creates a 'manifest' for the specified region, by reading directly from the HRegion object. 227 * This is used by the "online snapshot" when the table is enabled. 228 */ 229 public void addRegion(final HRegion region) throws IOException { 230 // Get the ManifestBuilder/RegionVisitor 231 RegionVisitor visitor = createRegionVisitor(desc); 232 233 // Visit the region and add it to the manifest 234 addRegion(region, visitor); 235 } 236 237 protected void addRegion(final HRegion region, RegionVisitor visitor) throws IOException { 238 // 1. dump region meta info into the snapshot directory 239 final String snapshotName = desc.getName(); 240 LOG.debug("Storing '" + region + "' region-info for snapshot=" + snapshotName); 241 Object regionData = visitor.regionOpen(region.getRegionInfo()); 242 monitor.rethrowException(); 243 244 // 2. iterate through all the stores in the region 245 LOG.debug("Creating references for hfiles"); 246 247 for (HStore store : region.getStores()) { 248 // 2.1. build the snapshot reference for the store 249 Object familyData = 250 visitor.familyOpen(regionData, store.getColumnFamilyDescriptor().getName()); 251 monitor.rethrowException(); 252 253 List<HStoreFile> storeFiles = new ArrayList<>(store.getStorefiles()); 254 if (LOG.isDebugEnabled()) { 255 LOG.debug("Adding snapshot references for " + storeFiles + " hfiles"); 256 } 257 258 // 2.2. iterate through all the store's files and create "references". 259 for (int i = 0, sz = storeFiles.size(); i < sz; i++) { 260 HStoreFile storeFile = storeFiles.get(i); 261 monitor.rethrowException(); 262 263 // create "reference" to this store file. 264 LOG.debug("Adding reference for file (" + (i + 1) + "/" + sz + "): " + storeFile.getPath() 265 + " for snapshot=" + snapshotName); 266 visitor.storeFile(regionData, familyData, storeFile.getFileInfo()); 267 } 268 visitor.familyClose(regionData, familyData); 269 } 270 visitor.regionClose(regionData); 271 } 272 273 /** 274 * Creates a 'manifest' for the specified region, by reading directly from the disk. This is used 275 * by the "offline snapshot" when the table is disabled. 276 */ 277 public void addRegion(final Path tableDir, final RegionInfo regionInfo) throws IOException { 278 // Get the ManifestBuilder/RegionVisitor 279 RegionVisitor visitor = createRegionVisitor(desc); 280 281 // Visit the region and add it to the manifest 282 addRegion(tableDir, regionInfo, visitor); 283 } 284 285 protected void addRegion(Path tableDir, RegionInfo regionInfo, RegionVisitor visitor) 286 throws IOException { 287 boolean isMobRegion = MobUtils.isMobRegionInfo(regionInfo); 288 try { 289 Path baseDir = tableDir; 290 // Open the RegionFS 291 if (isMobRegion) { 292 baseDir = CommonFSUtils.getTableDir(MobUtils.getMobHome(conf), regionInfo.getTable()); 293 } 294 HRegionFileSystem regionFs = 295 HRegionFileSystem.openRegionFromFileSystem(conf, rootFs, baseDir, regionInfo, true); 296 monitor.rethrowException(); 297 298 // 1. dump region meta info into the snapshot directory 299 LOG.debug("Storing region-info for snapshot."); 300 Object regionData = visitor.regionOpen(regionInfo); 301 monitor.rethrowException(); 302 303 // 2. iterate through all the stores in the region 304 LOG.debug("Creating references for hfiles"); 305 306 // This ensures that we have an atomic view of the directory as long as we have < ls limit 307 // (batch size of the files in a directory) on the namenode. Otherwise, we get back the files 308 // in batches and may miss files being added/deleted. This could be more robust (iteratively 309 // checking to see if we have all the files until we are sure), but the limit is currently 310 // 1000 files/batch, far more than the number of store files under a single column family. 311 for (ColumnFamilyDescriptor cfd : htd.getColumnFamilies()) { 312 Object familyData = visitor.familyOpen(regionData, cfd.getName()); 313 monitor.rethrowException(); 314 StoreFileTracker tracker = null; 315 if (isMobRegion) { 316 // MOB regions are always using the default SFT implementation 317 ColumnFamilyDescriptor defaultSFTCfd = ColumnFamilyDescriptorBuilder.newBuilder(cfd) 318 .setValue(StoreFileTrackerFactory.TRACKER_IMPL, 319 StoreFileTrackerFactory.Trackers.DEFAULT.name()) 320 .build(); 321 tracker = StoreFileTrackerFactory.create(conf, htd, defaultSFTCfd, regionFs); 322 } else { 323 tracker = StoreFileTrackerFactory.create(conf, htd, cfd, regionFs); 324 } 325 List<StoreFileInfo> storeFiles = tracker.load(); 326 if (storeFiles.isEmpty()) { 327 LOG.debug("No files under family: {}", cfd.getNameAsString()); 328 continue; 329 } 330 // 2.1. build the snapshot reference for the store 331 // iterate through all the store's files and create "references". 332 addReferenceFiles(visitor, regionData, familyData, storeFiles, false); 333 visitor.familyClose(regionData, familyData); 334 } 335 visitor.regionClose(regionData); 336 } catch (IOException e) { 337 // the mob directory might not be created yet, so do nothing when it is a mob region 338 if (!isMobRegion) { 339 throw e; 340 } 341 } 342 } 343 344 private List<StoreFileInfo> getStoreFiles(Path storeDir) throws IOException { 345 FileStatus[] stats = CommonFSUtils.listStatus(rootFs, storeDir); 346 if (stats == null) return null; 347 348 ArrayList<StoreFileInfo> storeFiles = new ArrayList<>(stats.length); 349 for (int i = 0; i < stats.length; ++i) { 350 storeFiles.add(new StoreFileInfo(conf, rootFs, stats[i])); 351 } 352 return storeFiles; 353 } 354 355 private void addReferenceFiles(RegionVisitor visitor, Object regionData, Object familyData, 356 Collection<StoreFileInfo> storeFiles, boolean isMob) throws IOException { 357 final String fileType = isMob ? "mob file" : "hfile"; 358 359 if (LOG.isDebugEnabled()) { 360 LOG.debug(String.format("Adding snapshot references for %s %ss", storeFiles, fileType)); 361 } 362 363 int i = 0; 364 int sz = storeFiles.size(); 365 for (StoreFileInfo storeFile : storeFiles) { 366 monitor.rethrowException(); 367 368 LOG.debug(String.format("Adding reference for %s (%d/%d): %s", fileType, ++i, sz, 369 storeFile.getPath())); 370 371 // create "reference" to this store file. 372 visitor.storeFile(regionData, familyData, storeFile); 373 } 374 } 375 376 /** 377 * Load the information in the SnapshotManifest. Called by SnapshotManifest.open() If the format 378 * is v2 and there is no data-manifest, means that we are loading an in-progress snapshot. Since 379 * we support rolling-upgrades, we loook for v1 and v2 regions format. 380 */ 381 private void load() throws IOException { 382 switch (getSnapshotFormat(desc)) { 383 case SnapshotManifestV1.DESCRIPTOR_VERSION: { 384 this.htd = FSTableDescriptors.getTableDescriptorFromFs(workingDirFs, workingDir); 385 ThreadPoolExecutor tpool = createExecutor("SnapshotManifestLoader"); 386 try { 387 this.regionManifests = 388 SnapshotManifestV1.loadRegionManifests(conf, tpool, rootFs, workingDir, desc); 389 } finally { 390 tpool.shutdown(); 391 } 392 break; 393 } 394 case SnapshotManifestV2.DESCRIPTOR_VERSION: { 395 SnapshotDataManifest dataManifest = readDataManifest(); 396 if (dataManifest != null) { 397 htd = ProtobufUtil.toTableDescriptor(dataManifest.getTableSchema()); 398 regionManifests = dataManifest.getRegionManifestsList(); 399 } else { 400 // Compatibility, load the v1 regions 401 // This happens only when the snapshot is in-progress and the cache wants to refresh. 402 List<SnapshotRegionManifest> v1Regions, v2Regions; 403 ThreadPoolExecutor tpool = createExecutor("SnapshotManifestLoader"); 404 try { 405 v1Regions = 406 SnapshotManifestV1.loadRegionManifests(conf, tpool, rootFs, workingDir, desc); 407 v2Regions = SnapshotManifestV2.loadRegionManifests(conf, tpool, rootFs, workingDir, 408 desc, manifestSizeLimit); 409 } catch (InvalidProtocolBufferException e) { 410 throw new CorruptedSnapshotException( 411 "unable to parse region manifest " + e.getMessage(), e); 412 } finally { 413 tpool.shutdown(); 414 } 415 if (v1Regions != null && v2Regions != null) { 416 regionManifests = new ArrayList<>(v1Regions.size() + v2Regions.size()); 417 regionManifests.addAll(v1Regions); 418 regionManifests.addAll(v2Regions); 419 } else if (v1Regions != null) { 420 regionManifests = v1Regions; 421 } else /* if (v2Regions != null) */ { 422 regionManifests = v2Regions; 423 } 424 } 425 break; 426 } 427 default: 428 throw new CorruptedSnapshotException("Invalid Snapshot version: " + desc.getVersion(), 429 ProtobufUtil.createSnapshotDesc(desc)); 430 } 431 } 432 433 /** 434 * Get the current snapshot working dir 435 */ 436 public Path getSnapshotDir() { 437 return this.workingDir; 438 } 439 440 /** 441 * Get the SnapshotDescription 442 */ 443 public SnapshotDescription getSnapshotDescription() { 444 return this.desc; 445 } 446 447 /** 448 * Get the table descriptor from the Snapshot 449 */ 450 public TableDescriptor getTableDescriptor() { 451 return this.htd; 452 } 453 454 /** 455 * Get all the Region Manifest from the snapshot 456 */ 457 public List<SnapshotRegionManifest> getRegionManifests() { 458 return this.regionManifests; 459 } 460 461 private void setStatusMsg(String msg) { 462 if (this.statusTask != null) { 463 statusTask.setStatus(msg); 464 } 465 } 466 467 /** 468 * Get all the Region Manifest from the snapshot. This is an helper to get a map with the region 469 * encoded name 470 */ 471 public Map<String, SnapshotRegionManifest> getRegionManifestsMap() { 472 if (regionManifests == null || regionManifests.isEmpty()) return null; 473 474 HashMap<String, SnapshotRegionManifest> regionsMap = new HashMap<>(regionManifests.size()); 475 for (SnapshotRegionManifest manifest : regionManifests) { 476 String regionName = getRegionNameFromManifest(manifest); 477 regionsMap.put(regionName, manifest); 478 } 479 return regionsMap; 480 } 481 482 public void consolidate() throws IOException { 483 if (getSnapshotFormat(desc) == SnapshotManifestV1.DESCRIPTOR_VERSION) { 484 LOG.info("Using old Snapshot Format"); 485 // write a copy of descriptor to the snapshot directory 486 FSTableDescriptors.createTableDescriptorForTableDirectory(workingDirFs, workingDir, htd, 487 false); 488 } else { 489 LOG.debug("Convert to Single Snapshot Manifest for {}", this.desc.getName()); 490 convertToV2SingleManifest(); 491 } 492 } 493 494 /* 495 * In case of rolling-upgrade, we try to read all the formats and build the snapshot with the 496 * latest format. 497 */ 498 private void convertToV2SingleManifest() throws IOException { 499 // Try to load v1 and v2 regions 500 List<SnapshotRegionManifest> v1Regions, v2Regions; 501 ThreadPoolExecutor tpool = createExecutor("SnapshotManifestLoader"); 502 setStatusMsg("Loading Region manifests for " + this.desc.getName()); 503 try { 504 v1Regions = 505 SnapshotManifestV1.loadRegionManifests(conf, tpool, workingDirFs, workingDir, desc); 506 v2Regions = SnapshotManifestV2.loadRegionManifests(conf, tpool, workingDirFs, workingDir, 507 desc, manifestSizeLimit); 508 509 SnapshotDataManifest.Builder dataManifestBuilder = SnapshotDataManifest.newBuilder(); 510 dataManifestBuilder.setTableSchema(ProtobufUtil.toTableSchema(htd)); 511 512 if (v1Regions != null && v1Regions.size() > 0) { 513 dataManifestBuilder.addAllRegionManifests(v1Regions); 514 } 515 if (v2Regions != null && v2Regions.size() > 0) { 516 dataManifestBuilder.addAllRegionManifests(v2Regions); 517 } 518 519 // Write the v2 Data Manifest. 520 // Once the data-manifest is written, the snapshot can be considered complete. 521 // Currently snapshots are written in a "temporary" directory and later 522 // moved to the "complated" snapshot directory. 523 setStatusMsg("Writing data manifest for " + this.desc.getName()); 524 SnapshotDataManifest dataManifest = dataManifestBuilder.build(); 525 writeDataManifest(dataManifest); 526 this.regionManifests = dataManifest.getRegionManifestsList(); 527 528 // Remove the region manifests. Everything is now in the data-manifest. 529 // The delete operation is "relaxed", unless we get an exception we keep going. 530 // The extra files in the snapshot directory will not give any problem, 531 // since they have the same content as the data manifest, and even by re-reading 532 // them we will get the same information. 533 int totalDeletes = 0; 534 ExecutorCompletionService<Void> completionService = new ExecutorCompletionService<>(tpool); 535 if (v1Regions != null) { 536 for (SnapshotRegionManifest regionManifest : v1Regions) { 537 ++totalDeletes; 538 completionService.submit(() -> { 539 SnapshotManifestV1.deleteRegionManifest(workingDirFs, workingDir, regionManifest); 540 return null; 541 }); 542 } 543 } 544 if (v2Regions != null) { 545 for (SnapshotRegionManifest regionManifest : v2Regions) { 546 ++totalDeletes; 547 completionService.submit(() -> { 548 SnapshotManifestV2.deleteRegionManifest(workingDirFs, workingDir, regionManifest); 549 return null; 550 }); 551 } 552 } 553 // Wait for the deletes to finish. 554 for (int i = 0; i < totalDeletes; i++) { 555 try { 556 completionService.take().get(); 557 } catch (InterruptedException ie) { 558 throw new InterruptedIOException(ie.getMessage()); 559 } catch (ExecutionException e) { 560 throw new IOException("Error deleting region manifests", e.getCause()); 561 } 562 } 563 } finally { 564 tpool.shutdown(); 565 } 566 } 567 568 /* 569 * Write the SnapshotDataManifest file 570 */ 571 private void writeDataManifest(final SnapshotDataManifest manifest) throws IOException { 572 try ( 573 FSDataOutputStream stream = workingDirFs.create(new Path(workingDir, DATA_MANIFEST_NAME))) { 574 manifest.writeTo(stream); 575 } 576 } 577 578 /* 579 * Read the SnapshotDataManifest file 580 */ 581 private SnapshotDataManifest readDataManifest() throws IOException { 582 try (FSDataInputStream in = workingDirFs.open(new Path(workingDir, DATA_MANIFEST_NAME))) { 583 CodedInputStream cin = CodedInputStream.newInstance(in); 584 cin.setSizeLimit(manifestSizeLimit); 585 return SnapshotDataManifest.parseFrom(cin); 586 } catch (FileNotFoundException e) { 587 return null; 588 } catch (InvalidProtocolBufferException e) { 589 throw new CorruptedSnapshotException("unable to parse data manifest " + e.getMessage(), e); 590 } 591 } 592 593 private ThreadPoolExecutor createExecutor(final String name) { 594 return createExecutor(conf, name); 595 } 596 597 public static ThreadPoolExecutor createExecutor(final Configuration conf, final String name) { 598 int maxThreads = conf.getInt("hbase.snapshot.thread.pool.max", 8); 599 return Threads.getBoundedCachedThreadPool(maxThreads, 30L, TimeUnit.SECONDS, 600 new ThreadFactoryBuilder().setNameFormat(name + "-pool-%d").setDaemon(true) 601 .setUncaughtExceptionHandler(Threads.LOGGING_EXCEPTION_HANDLER).build()); 602 } 603 604 /** 605 * Extract the region encoded name from the region manifest 606 */ 607 static String getRegionNameFromManifest(final SnapshotRegionManifest manifest) { 608 byte[] regionName = 609 RegionInfo.createRegionName(ProtobufUtil.toTableName(manifest.getRegionInfo().getTableName()), 610 manifest.getRegionInfo().getStartKey().toByteArray(), 611 manifest.getRegionInfo().getRegionId(), true); 612 return RegionInfo.encodeRegionName(regionName); 613 } 614 615 /* 616 * Return the snapshot format 617 */ 618 private static int getSnapshotFormat(final SnapshotDescription desc) { 619 return desc.hasVersion() ? desc.getVersion() : SnapshotManifestV1.DESCRIPTOR_VERSION; 620 } 621}