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