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.IOException; 022import java.io.InputStream; 023import java.io.OutputStream; 024import java.util.ArrayList; 025import java.util.Arrays; 026import java.util.Collections; 027import java.util.HashMap; 028import java.util.HashSet; 029import java.util.LinkedList; 030import java.util.List; 031import java.util.Map; 032import java.util.Map.Entry; 033import java.util.Set; 034import java.util.TreeMap; 035import java.util.concurrent.ThreadPoolExecutor; 036 037import org.apache.hadoop.conf.Configuration; 038import org.apache.hadoop.fs.FileStatus; 039import org.apache.hadoop.fs.FileSystem; 040import org.apache.hadoop.fs.Path; 041import org.apache.hadoop.hbase.MetaTableAccessor; 042import org.apache.hadoop.hbase.TableName; 043import org.apache.hadoop.hbase.backup.HFileArchiver; 044import org.apache.hadoop.hbase.client.Connection; 045import org.apache.hadoop.hbase.client.ConnectionFactory; 046import org.apache.hadoop.hbase.client.RegionInfo; 047import org.apache.hadoop.hbase.client.RegionInfoBuilder; 048import org.apache.hadoop.hbase.client.TableDescriptor; 049import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher; 050import org.apache.hadoop.hbase.io.HFileLink; 051import org.apache.hadoop.hbase.io.Reference; 052import org.apache.hadoop.hbase.mob.MobUtils; 053import org.apache.hadoop.hbase.monitoring.MonitoredTask; 054import org.apache.hadoop.hbase.monitoring.TaskMonitor; 055import org.apache.hadoop.hbase.regionserver.HRegion; 056import org.apache.hadoop.hbase.regionserver.HRegionFileSystem; 057import org.apache.hadoop.hbase.regionserver.StoreFileInfo; 058import org.apache.hadoop.hbase.security.access.AccessControlClient; 059import org.apache.hadoop.hbase.security.access.Permission; 060import org.apache.hadoop.hbase.security.access.ShadedAccessControlUtil; 061import org.apache.hadoop.hbase.security.access.TablePermission; 062import org.apache.hadoop.hbase.util.Bytes; 063import org.apache.hadoop.hbase.util.FSUtils; 064import org.apache.hadoop.hbase.util.ModifyRegionUtils; 065import org.apache.hadoop.hbase.util.Pair; 066import org.apache.hadoop.io.IOUtils; 067import org.apache.yetus.audience.InterfaceAudience; 068import org.slf4j.Logger; 069import org.slf4j.LoggerFactory; 070import org.apache.hbase.thirdparty.com.google.common.collect.ListMultimap; 071import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 072import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription; 073import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest; 074 075/** 076 * Helper to Restore/Clone a Snapshot 077 * 078 * <p>The helper assumes that a table is already created, and by calling restore() 079 * the content present in the snapshot will be restored as the new content of the table. 080 * 081 * <p>Clone from Snapshot: If the target table is empty, the restore operation 082 * is just a "clone operation", where the only operations are: 083 * <ul> 084 * <li>for each region in the snapshot create a new region 085 * (note that the region will have a different name, since the encoding contains the table name) 086 * <li>for each file in the region create a new HFileLink to point to the original file. 087 * <li>restore the logs, if any 088 * </ul> 089 * 090 * <p>Restore from Snapshot: 091 * <ul> 092 * <li>for each region in the table verify which are available in the snapshot and which are not 093 * <ul> 094 * <li>if the region is not present in the snapshot, remove it. 095 * <li>if the region is present in the snapshot 096 * <ul> 097 * <li>for each file in the table region verify which are available in the snapshot 098 * <ul> 099 * <li>if the hfile is not present in the snapshot, remove it 100 * <li>if the hfile is present, keep it (nothing to do) 101 * </ul> 102 * <li>for each file in the snapshot region but not in the table 103 * <ul> 104 * <li>create a new HFileLink that point to the original file 105 * </ul> 106 * </ul> 107 * </ul> 108 * <li>for each region in the snapshot not present in the current table state 109 * <ul> 110 * <li>create a new region and for each file in the region create a new HFileLink 111 * (This is the same as the clone operation) 112 * </ul> 113 * <li>restore the logs, if any 114 * </ul> 115 */ 116@InterfaceAudience.Private 117public class RestoreSnapshotHelper { 118 private static final Logger LOG = LoggerFactory.getLogger(RestoreSnapshotHelper.class); 119 120 private final Map<byte[], byte[]> regionsMap = new TreeMap<>(Bytes.BYTES_COMPARATOR); 121 122 private final Map<String, Pair<String, String> > parentsMap = new HashMap<>(); 123 124 private final ForeignExceptionDispatcher monitor; 125 private final MonitoredTask status; 126 127 private final SnapshotManifest snapshotManifest; 128 private final SnapshotDescription snapshotDesc; 129 private final TableName snapshotTable; 130 131 private final TableDescriptor tableDesc; 132 private final Path rootDir; 133 private final Path tableDir; 134 135 private final Configuration conf; 136 private final FileSystem fs; 137 private final boolean createBackRefs; 138 139 public RestoreSnapshotHelper(final Configuration conf, 140 final FileSystem fs, 141 final SnapshotManifest manifest, 142 final TableDescriptor tableDescriptor, 143 final Path rootDir, 144 final ForeignExceptionDispatcher monitor, 145 final MonitoredTask status) { 146 this(conf, fs, manifest, tableDescriptor, rootDir, monitor, status, true); 147 } 148 149 public RestoreSnapshotHelper(final Configuration conf, 150 final FileSystem fs, 151 final SnapshotManifest manifest, 152 final TableDescriptor tableDescriptor, 153 final Path rootDir, 154 final ForeignExceptionDispatcher monitor, 155 final MonitoredTask status, 156 final boolean createBackRefs) 157 { 158 this.fs = fs; 159 this.conf = conf; 160 this.snapshotManifest = manifest; 161 this.snapshotDesc = manifest.getSnapshotDescription(); 162 this.snapshotTable = TableName.valueOf(snapshotDesc.getTable()); 163 this.tableDesc = tableDescriptor; 164 this.rootDir = rootDir; 165 this.tableDir = FSUtils.getTableDir(rootDir, tableDesc.getTableName()); 166 this.monitor = monitor; 167 this.status = status; 168 this.createBackRefs = createBackRefs; 169 } 170 171 /** 172 * Restore the on-disk table to a specified snapshot state. 173 * @return the set of regions touched by the restore operation 174 */ 175 public RestoreMetaChanges restoreHdfsRegions() throws IOException { 176 ThreadPoolExecutor exec = SnapshotManifest.createExecutor(conf, "RestoreSnapshot"); 177 try { 178 return restoreHdfsRegions(exec); 179 } finally { 180 exec.shutdown(); 181 } 182 } 183 184 private RestoreMetaChanges restoreHdfsRegions(final ThreadPoolExecutor exec) throws IOException { 185 LOG.info("starting restore table regions using snapshot=" + snapshotDesc); 186 187 Map<String, SnapshotRegionManifest> regionManifests = snapshotManifest.getRegionManifestsMap(); 188 if (regionManifests == null) { 189 LOG.warn("Nothing to restore. Snapshot " + snapshotDesc + " looks empty"); 190 return null; 191 } 192 193 RestoreMetaChanges metaChanges = new RestoreMetaChanges(tableDesc, parentsMap); 194 195 // Take a copy of the manifest.keySet() since we are going to modify 196 // this instance, by removing the regions already present in the restore dir. 197 Set<String> regionNames = new HashSet<>(regionManifests.keySet()); 198 199 List<RegionInfo> tableRegions = getTableRegions(); 200 201 RegionInfo mobRegion = MobUtils.getMobRegionInfo(snapshotManifest.getTableDescriptor() 202 .getTableName()); 203 if (tableRegions != null) { 204 // restore the mob region in case 205 if (regionNames.contains(mobRegion.getEncodedName())) { 206 monitor.rethrowException(); 207 status.setStatus("Restoring mob region..."); 208 List<RegionInfo> mobRegions = new ArrayList<>(1); 209 mobRegions.add(mobRegion); 210 restoreHdfsMobRegions(exec, regionManifests, mobRegions); 211 regionNames.remove(mobRegion.getEncodedName()); 212 status.setStatus("Finished restoring mob region."); 213 } 214 } 215 if (regionNames.contains(mobRegion.getEncodedName())) { 216 // add the mob region 217 monitor.rethrowException(); 218 status.setStatus("Cloning mob region..."); 219 cloneHdfsMobRegion(regionManifests, mobRegion); 220 regionNames.remove(mobRegion.getEncodedName()); 221 status.setStatus("Finished cloning mob region."); 222 } 223 224 // Identify which region are still available and which not. 225 // NOTE: we rely upon the region name as: "table name, start key, end key" 226 if (tableRegions != null) { 227 monitor.rethrowException(); 228 for (RegionInfo regionInfo: tableRegions) { 229 String regionName = regionInfo.getEncodedName(); 230 if (regionNames.contains(regionName)) { 231 LOG.info("region to restore: " + regionName); 232 regionNames.remove(regionName); 233 metaChanges.addRegionToRestore(ProtobufUtil.toRegionInfo(regionManifests.get(regionName) 234 .getRegionInfo())); 235 } else { 236 LOG.info("region to remove: " + regionName); 237 metaChanges.addRegionToRemove(regionInfo); 238 } 239 } 240 } 241 242 // Regions to Add: present in the snapshot but not in the current table 243 List<RegionInfo> regionsToAdd = new ArrayList<>(regionNames.size()); 244 if (regionNames.size() > 0) { 245 monitor.rethrowException(); 246 for (String regionName: regionNames) { 247 LOG.info("region to add: " + regionName); 248 regionsToAdd.add(ProtobufUtil.toRegionInfo(regionManifests.get(regionName) 249 .getRegionInfo())); 250 } 251 } 252 253 // Create new regions cloning from the snapshot 254 // HBASE-19980: We need to call cloneHdfsRegions() before restoreHdfsRegions() because 255 // regionsMap is constructed in cloneHdfsRegions() and it can be used in restoreHdfsRegions(). 256 monitor.rethrowException(); 257 status.setStatus("Cloning regions..."); 258 RegionInfo[] clonedRegions = cloneHdfsRegions(exec, regionManifests, regionsToAdd); 259 metaChanges.setNewRegions(clonedRegions); 260 status.setStatus("Finished cloning regions."); 261 262 // Restore regions using the snapshot data 263 monitor.rethrowException(); 264 status.setStatus("Restoring table regions..."); 265 restoreHdfsRegions(exec, regionManifests, metaChanges.getRegionsToRestore()); 266 status.setStatus("Finished restoring all table regions."); 267 268 // Remove regions from the current table 269 monitor.rethrowException(); 270 status.setStatus("Starting to delete excess regions from table"); 271 removeHdfsRegions(exec, metaChanges.getRegionsToRemove()); 272 status.setStatus("Finished deleting excess regions from table."); 273 274 LOG.info("finishing restore table regions using snapshot=" + snapshotDesc); 275 276 return metaChanges; 277 } 278 279 /** 280 * Describe the set of operations needed to update hbase:meta after restore. 281 */ 282 public static class RestoreMetaChanges { 283 private final Map<String, Pair<String, String> > parentsMap; 284 private final TableDescriptor htd; 285 286 private List<RegionInfo> regionsToRestore = null; 287 private List<RegionInfo> regionsToRemove = null; 288 private List<RegionInfo> regionsToAdd = null; 289 290 public RestoreMetaChanges(TableDescriptor htd, Map<String, Pair<String, String> > parentsMap) { 291 this.parentsMap = parentsMap; 292 this.htd = htd; 293 } 294 295 public TableDescriptor getTableDescriptor() { 296 return htd; 297 } 298 299 /** 300 * Returns the map of parent-children_pair. 301 * @return the map 302 */ 303 public Map<String, Pair<String, String>> getParentToChildrenPairMap() { 304 return this.parentsMap; 305 } 306 307 /** 308 * @return true if there're new regions 309 */ 310 public boolean hasRegionsToAdd() { 311 return this.regionsToAdd != null && this.regionsToAdd.size() > 0; 312 } 313 314 /** 315 * Returns the list of new regions added during the on-disk restore. 316 * The caller is responsible to add the regions to META. 317 * e.g MetaTableAccessor.addRegionsToMeta(...) 318 * @return the list of regions to add to META 319 */ 320 public List<RegionInfo> getRegionsToAdd() { 321 return this.regionsToAdd; 322 } 323 324 /** 325 * @return true if there're regions to restore 326 */ 327 public boolean hasRegionsToRestore() { 328 return this.regionsToRestore != null && this.regionsToRestore.size() > 0; 329 } 330 331 /** 332 * Returns the list of 'restored regions' during the on-disk restore. 333 * The caller is responsible to add the regions to hbase:meta if not present. 334 * @return the list of regions restored 335 */ 336 public List<RegionInfo> getRegionsToRestore() { 337 return this.regionsToRestore; 338 } 339 340 /** 341 * @return true if there're regions to remove 342 */ 343 public boolean hasRegionsToRemove() { 344 return this.regionsToRemove != null && this.regionsToRemove.size() > 0; 345 } 346 347 /** 348 * Returns the list of regions removed during the on-disk restore. 349 * The caller is responsible to remove the regions from META. 350 * e.g. MetaTableAccessor.deleteRegions(...) 351 * @return the list of regions to remove from META 352 */ 353 public List<RegionInfo> getRegionsToRemove() { 354 return this.regionsToRemove; 355 } 356 357 void setNewRegions(final RegionInfo[] hris) { 358 if (hris != null) { 359 regionsToAdd = Arrays.asList(hris); 360 } else { 361 regionsToAdd = null; 362 } 363 } 364 365 void addRegionToRemove(final RegionInfo hri) { 366 if (regionsToRemove == null) { 367 regionsToRemove = new LinkedList<>(); 368 } 369 regionsToRemove.add(hri); 370 } 371 372 void addRegionToRestore(final RegionInfo hri) { 373 if (regionsToRestore == null) { 374 regionsToRestore = new LinkedList<>(); 375 } 376 regionsToRestore.add(hri); 377 } 378 379 public void updateMetaParentRegions(Connection connection, 380 final List<RegionInfo> regionInfos) throws IOException { 381 if (regionInfos == null || parentsMap.isEmpty()) return; 382 383 // Extract region names and offlined regions 384 Map<String, RegionInfo> regionsByName = new HashMap<>(regionInfos.size()); 385 List<RegionInfo> parentRegions = new LinkedList<>(); 386 for (RegionInfo regionInfo: regionInfos) { 387 if (regionInfo.isSplitParent()) { 388 parentRegions.add(regionInfo); 389 } else { 390 regionsByName.put(regionInfo.getEncodedName(), regionInfo); 391 } 392 } 393 394 // Update Offline parents 395 for (RegionInfo regionInfo: parentRegions) { 396 Pair<String, String> daughters = parentsMap.get(regionInfo.getEncodedName()); 397 if (daughters == null) { 398 // The snapshot contains an unreferenced region. 399 // It will be removed by the CatalogJanitor. 400 LOG.warn("Skip update of unreferenced offline parent: " + regionInfo); 401 continue; 402 } 403 404 // One side of the split is already compacted 405 if (daughters.getSecond() == null) { 406 daughters.setSecond(daughters.getFirst()); 407 } 408 409 LOG.debug("Update splits parent " + regionInfo.getEncodedName() + " -> " + daughters); 410 MetaTableAccessor.addSplitsToParent(connection, regionInfo, 411 regionsByName.get(daughters.getFirst()), 412 regionsByName.get(daughters.getSecond())); 413 } 414 } 415 } 416 417 /** 418 * Remove specified regions from the file-system, using the archiver. 419 */ 420 private void removeHdfsRegions(final ThreadPoolExecutor exec, final List<RegionInfo> regions) 421 throws IOException { 422 if (regions == null || regions.isEmpty()) return; 423 ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() { 424 @Override 425 public void editRegion(final RegionInfo hri) throws IOException { 426 HFileArchiver.archiveRegion(conf, fs, hri); 427 } 428 }); 429 } 430 431 /** 432 * Restore specified regions by restoring content to the snapshot state. 433 */ 434 private void restoreHdfsRegions(final ThreadPoolExecutor exec, 435 final Map<String, SnapshotRegionManifest> regionManifests, 436 final List<RegionInfo> regions) throws IOException { 437 if (regions == null || regions.isEmpty()) return; 438 ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() { 439 @Override 440 public void editRegion(final RegionInfo hri) throws IOException { 441 restoreRegion(hri, regionManifests.get(hri.getEncodedName())); 442 } 443 }); 444 } 445 446 /** 447 * Restore specified mob regions by restoring content to the snapshot state. 448 */ 449 private void restoreHdfsMobRegions(final ThreadPoolExecutor exec, 450 final Map<String, SnapshotRegionManifest> regionManifests, 451 final List<RegionInfo> regions) throws IOException { 452 if (regions == null || regions.isEmpty()) return; 453 ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() { 454 @Override 455 public void editRegion(final RegionInfo hri) throws IOException { 456 restoreMobRegion(hri, regionManifests.get(hri.getEncodedName())); 457 } 458 }); 459 } 460 461 private Map<String, List<SnapshotRegionManifest.StoreFile>> getRegionHFileReferences( 462 final SnapshotRegionManifest manifest) { 463 Map<String, List<SnapshotRegionManifest.StoreFile>> familyMap = 464 new HashMap<>(manifest.getFamilyFilesCount()); 465 for (SnapshotRegionManifest.FamilyFiles familyFiles: manifest.getFamilyFilesList()) { 466 familyMap.put(familyFiles.getFamilyName().toStringUtf8(), 467 new ArrayList<>(familyFiles.getStoreFilesList())); 468 } 469 return familyMap; 470 } 471 472 /** 473 * Restore region by removing files not in the snapshot 474 * and adding the missing ones from the snapshot. 475 */ 476 private void restoreRegion(final RegionInfo regionInfo, 477 final SnapshotRegionManifest regionManifest) throws IOException { 478 restoreRegion(regionInfo, regionManifest, new Path(tableDir, regionInfo.getEncodedName())); 479 } 480 481 /** 482 * Restore mob region by removing files not in the snapshot 483 * and adding the missing ones from the snapshot. 484 */ 485 private void restoreMobRegion(final RegionInfo regionInfo, 486 final SnapshotRegionManifest regionManifest) throws IOException { 487 if (regionManifest == null) { 488 return; 489 } 490 restoreRegion(regionInfo, regionManifest, 491 MobUtils.getMobRegionPath(conf, tableDesc.getTableName())); 492 } 493 494 /** 495 * Restore region by removing files not in the snapshot 496 * and adding the missing ones from the snapshot. 497 */ 498 private void restoreRegion(final RegionInfo regionInfo, 499 final SnapshotRegionManifest regionManifest, Path regionDir) throws IOException { 500 Map<String, List<SnapshotRegionManifest.StoreFile>> snapshotFiles = 501 getRegionHFileReferences(regionManifest); 502 503 String tableName = tableDesc.getTableName().getNameAsString(); 504 final String snapshotName = snapshotDesc.getName(); 505 506 // Restore families present in the table 507 for (Path familyDir: FSUtils.getFamilyDirs(fs, regionDir)) { 508 byte[] family = Bytes.toBytes(familyDir.getName()); 509 Set<String> familyFiles = getTableRegionFamilyFiles(familyDir); 510 List<SnapshotRegionManifest.StoreFile> snapshotFamilyFiles = 511 snapshotFiles.remove(familyDir.getName()); 512 if (snapshotFamilyFiles != null) { 513 List<SnapshotRegionManifest.StoreFile> hfilesToAdd = new ArrayList<>(); 514 for (SnapshotRegionManifest.StoreFile storeFile: snapshotFamilyFiles) { 515 if (familyFiles.contains(storeFile.getName())) { 516 // HFile already present 517 familyFiles.remove(storeFile.getName()); 518 } else { 519 // HFile missing 520 hfilesToAdd.add(storeFile); 521 } 522 } 523 524 // Remove hfiles not present in the snapshot 525 for (String hfileName: familyFiles) { 526 Path hfile = new Path(familyDir, hfileName); 527 LOG.trace("Removing HFile=" + hfileName + " not present in snapshot=" + snapshotName+ 528 " from region=" + regionInfo.getEncodedName() + " table=" + tableName); 529 HFileArchiver.archiveStoreFile(conf, fs, regionInfo, tableDir, family, hfile); 530 } 531 532 // Restore Missing files 533 for (SnapshotRegionManifest.StoreFile storeFile: hfilesToAdd) { 534 LOG.debug("Restoring missing HFileLink " + storeFile.getName() + 535 " of snapshot=" + snapshotName+ 536 " to region=" + regionInfo.getEncodedName() + " table=" + tableName); 537 restoreStoreFile(familyDir, regionInfo, storeFile, createBackRefs); 538 } 539 } else { 540 // Family doesn't exists in the snapshot 541 LOG.trace("Removing family=" + Bytes.toString(family) + " in snapshot=" + snapshotName + 542 " from region=" + regionInfo.getEncodedName() + " table=" + tableName); 543 HFileArchiver.archiveFamilyByFamilyDir(fs, conf, regionInfo, familyDir, family); 544 fs.delete(familyDir, true); 545 } 546 } 547 548 // Add families not present in the table 549 for (Map.Entry<String, List<SnapshotRegionManifest.StoreFile>> familyEntry: 550 snapshotFiles.entrySet()) { 551 Path familyDir = new Path(regionDir, familyEntry.getKey()); 552 if (!fs.mkdirs(familyDir)) { 553 throw new IOException("Unable to create familyDir=" + familyDir); 554 } 555 556 for (SnapshotRegionManifest.StoreFile storeFile: familyEntry.getValue()) { 557 LOG.trace("Adding HFileLink (Not present in the table) " + storeFile.getName() 558 + " of snapshot " + snapshotName + " to table=" + tableName); 559 restoreStoreFile(familyDir, regionInfo, storeFile, createBackRefs); 560 } 561 } 562 } 563 564 /** 565 * @return The set of files in the specified family directory. 566 */ 567 private Set<String> getTableRegionFamilyFiles(final Path familyDir) throws IOException { 568 FileStatus[] hfiles = FSUtils.listStatus(fs, familyDir); 569 if (hfiles == null) return Collections.emptySet(); 570 571 Set<String> familyFiles = new HashSet<>(hfiles.length); 572 for (int i = 0; i < hfiles.length; ++i) { 573 String hfileName = hfiles[i].getPath().getName(); 574 familyFiles.add(hfileName); 575 } 576 577 return familyFiles; 578 } 579 580 /** 581 * Clone specified regions. For each region create a new region 582 * and create a HFileLink for each hfile. 583 */ 584 private RegionInfo[] cloneHdfsRegions(final ThreadPoolExecutor exec, 585 final Map<String, SnapshotRegionManifest> regionManifests, 586 final List<RegionInfo> regions) throws IOException { 587 if (regions == null || regions.isEmpty()) return null; 588 589 final Map<String, RegionInfo> snapshotRegions = new HashMap<>(regions.size()); 590 final String snapshotName = snapshotDesc.getName(); 591 592 // clone region info (change embedded tableName with the new one) 593 RegionInfo[] clonedRegionsInfo = new RegionInfo[regions.size()]; 594 for (int i = 0; i < clonedRegionsInfo.length; ++i) { 595 // clone the region info from the snapshot region info 596 RegionInfo snapshotRegionInfo = regions.get(i); 597 clonedRegionsInfo[i] = cloneRegionInfo(snapshotRegionInfo); 598 599 // add the region name mapping between snapshot and cloned 600 String snapshotRegionName = snapshotRegionInfo.getEncodedName(); 601 String clonedRegionName = clonedRegionsInfo[i].getEncodedName(); 602 regionsMap.put(Bytes.toBytes(snapshotRegionName), Bytes.toBytes(clonedRegionName)); 603 LOG.info("clone region=" + snapshotRegionName + " as " + clonedRegionName + 604 " in snapshot " + snapshotName); 605 606 // Add mapping between cloned region name and snapshot region info 607 snapshotRegions.put(clonedRegionName, snapshotRegionInfo); 608 } 609 610 // create the regions on disk 611 ModifyRegionUtils.createRegions(exec, conf, rootDir, 612 tableDesc, clonedRegionsInfo, new ModifyRegionUtils.RegionFillTask() { 613 @Override 614 public void fillRegion(final HRegion region) throws IOException { 615 RegionInfo snapshotHri = snapshotRegions.get(region.getRegionInfo().getEncodedName()); 616 cloneRegion(region, snapshotHri, regionManifests.get(snapshotHri.getEncodedName())); 617 } 618 }); 619 620 return clonedRegionsInfo; 621 } 622 623 /** 624 * Clone the mob region. For the region create a new region 625 * and create a HFileLink for each hfile. 626 */ 627 private void cloneHdfsMobRegion(final Map<String, SnapshotRegionManifest> regionManifests, 628 final RegionInfo region) throws IOException { 629 // clone region info (change embedded tableName with the new one) 630 Path clonedRegionPath = MobUtils.getMobRegionPath(rootDir, tableDesc.getTableName()); 631 cloneRegion(clonedRegionPath, region, regionManifests.get(region.getEncodedName())); 632 } 633 634 /** 635 * Clone region directory content from the snapshot info. 636 * 637 * Each region is encoded with the table name, so the cloned region will have 638 * a different region name. 639 * 640 * Instead of copying the hfiles a HFileLink is created. 641 * 642 * @param regionDir {@link Path} cloned dir 643 * @param snapshotRegionInfo 644 */ 645 private void cloneRegion(final Path regionDir, final RegionInfo snapshotRegionInfo, 646 final SnapshotRegionManifest manifest) throws IOException { 647 final String tableName = tableDesc.getTableName().getNameAsString(); 648 final String snapshotName = snapshotDesc.getName(); 649 for (SnapshotRegionManifest.FamilyFiles familyFiles: manifest.getFamilyFilesList()) { 650 Path familyDir = new Path(regionDir, familyFiles.getFamilyName().toStringUtf8()); 651 for (SnapshotRegionManifest.StoreFile storeFile: familyFiles.getStoreFilesList()) { 652 LOG.info("Adding HFileLink " + storeFile.getName() +" from cloned region " 653 + "in snapshot " + snapshotName + " to table=" + tableName); 654 restoreStoreFile(familyDir, snapshotRegionInfo, storeFile, createBackRefs); 655 } 656 } 657 } 658 659 /** 660 * Clone region directory content from the snapshot info. 661 * 662 * Each region is encoded with the table name, so the cloned region will have 663 * a different region name. 664 * 665 * Instead of copying the hfiles a HFileLink is created. 666 * 667 * @param region {@link HRegion} cloned 668 * @param snapshotRegionInfo 669 */ 670 private void cloneRegion(final HRegion region, final RegionInfo snapshotRegionInfo, 671 final SnapshotRegionManifest manifest) throws IOException { 672 cloneRegion(new Path(tableDir, region.getRegionInfo().getEncodedName()), snapshotRegionInfo, 673 manifest); 674 } 675 676 /** 677 * Create a new {@link HFileLink} to reference the store file. 678 * <p>The store file in the snapshot can be a simple hfile, an HFileLink or a reference. 679 * <ul> 680 * <li>hfile: abc -> table=region-abc 681 * <li>reference: abc.1234 -> table=region-abc.1234 682 * <li>hfilelink: table=region-hfile -> table=region-hfile 683 * </ul> 684 * @param familyDir destination directory for the store file 685 * @param regionInfo destination region info for the table 686 * @param createBackRef - Whether back reference should be created. Defaults to true. 687 * @param storeFile store file name (can be a Reference, HFileLink or simple HFile) 688 */ 689 private void restoreStoreFile(final Path familyDir, final RegionInfo regionInfo, 690 final SnapshotRegionManifest.StoreFile storeFile, final boolean createBackRef) 691 throws IOException { 692 String hfileName = storeFile.getName(); 693 if (HFileLink.isHFileLink(hfileName)) { 694 HFileLink.createFromHFileLink(conf, fs, familyDir, hfileName, createBackRef); 695 } else if (StoreFileInfo.isReference(hfileName)) { 696 restoreReferenceFile(familyDir, regionInfo, storeFile); 697 } else { 698 HFileLink.create(conf, fs, familyDir, regionInfo, hfileName, createBackRef); 699 } 700 } 701 702 /** 703 * Create a new {@link Reference} as copy of the source one. 704 * <p><blockquote><pre> 705 * The source table looks like: 706 * 1234/abc (original file) 707 * 5678/abc.1234 (reference file) 708 * 709 * After the clone operation looks like: 710 * wxyz/table=1234-abc 711 * stuv/table=1234-abc.wxyz 712 * 713 * NOTE that the region name in the clone changes (md5 of regioninfo) 714 * and the reference should reflect that change. 715 * </pre></blockquote> 716 * @param familyDir destination directory for the store file 717 * @param regionInfo destination region info for the table 718 * @param storeFile reference file name 719 */ 720 private void restoreReferenceFile(final Path familyDir, final RegionInfo regionInfo, 721 final SnapshotRegionManifest.StoreFile storeFile) throws IOException { 722 String hfileName = storeFile.getName(); 723 724 // Extract the referred information (hfile name and parent region) 725 Path refPath = 726 StoreFileInfo.getReferredToFile(new Path(new Path(new Path(new Path(snapshotTable 727 .getNamespaceAsString(), snapshotTable.getQualifierAsString()), regionInfo 728 .getEncodedName()), familyDir.getName()), hfileName)); 729 String snapshotRegionName = refPath.getParent().getParent().getName(); 730 String fileName = refPath.getName(); 731 732 // The new reference should have the cloned region name as parent, if it is a clone. 733 String clonedRegionName = Bytes.toString(regionsMap.get(Bytes.toBytes(snapshotRegionName))); 734 if (clonedRegionName == null) clonedRegionName = snapshotRegionName; 735 736 // The output file should be a reference link table=snapshotRegion-fileName.clonedRegionName 737 Path linkPath = null; 738 String refLink = fileName; 739 if (!HFileLink.isHFileLink(fileName)) { 740 refLink = HFileLink.createHFileLinkName(snapshotTable, snapshotRegionName, fileName); 741 linkPath = new Path(familyDir, 742 HFileLink.createHFileLinkName(snapshotTable, regionInfo.getEncodedName(), hfileName)); 743 } 744 745 Path outPath = new Path(familyDir, refLink + '.' + clonedRegionName); 746 747 // Create the new reference 748 if (storeFile.hasReference()) { 749 Reference reference = Reference.convert(storeFile.getReference()); 750 reference.write(fs, outPath); 751 } else { 752 InputStream in; 753 if (linkPath != null) { 754 in = HFileLink.buildFromHFileLinkPattern(conf, linkPath).open(fs); 755 } else { 756 linkPath = new Path(new Path(HRegion.getRegionDir(snapshotManifest.getSnapshotDir(), 757 regionInfo.getEncodedName()), familyDir.getName()), hfileName); 758 in = fs.open(linkPath); 759 } 760 OutputStream out = fs.create(outPath); 761 IOUtils.copyBytes(in, out, conf); 762 } 763 764 // Add the daughter region to the map 765 String regionName = Bytes.toString(regionsMap.get(regionInfo.getEncodedNameAsBytes())); 766 if (regionName == null) { 767 regionName = regionInfo.getEncodedName(); 768 } 769 LOG.debug("Restore reference " + regionName + " to " + clonedRegionName); 770 synchronized (parentsMap) { 771 Pair<String, String> daughters = parentsMap.get(clonedRegionName); 772 if (daughters == null) { 773 // In case one side of the split is already compacted, regionName is put as both first and 774 // second of Pair 775 daughters = new Pair<>(regionName, regionName); 776 parentsMap.put(clonedRegionName, daughters); 777 } else if (!regionName.equals(daughters.getFirst())) { 778 daughters.setSecond(regionName); 779 } 780 } 781 } 782 783 /** 784 * Create a new {@link RegionInfo} from the snapshot region info. 785 * Keep the same startKey, endKey, regionId and split information but change 786 * the table name. 787 * 788 * @param snapshotRegionInfo Info for region to clone. 789 * @return the new HRegion instance 790 */ 791 public RegionInfo cloneRegionInfo(final RegionInfo snapshotRegionInfo) { 792 return cloneRegionInfo(tableDesc.getTableName(), snapshotRegionInfo); 793 } 794 795 public static RegionInfo cloneRegionInfo(TableName tableName, RegionInfo snapshotRegionInfo) { 796 return RegionInfoBuilder.newBuilder(tableName) 797 .setStartKey(snapshotRegionInfo.getStartKey()) 798 .setEndKey(snapshotRegionInfo.getEndKey()) 799 .setSplit(snapshotRegionInfo.isSplit()) 800 .setRegionId(snapshotRegionInfo.getRegionId()) 801 .setOffline(snapshotRegionInfo.isOffline()) 802 .build(); 803 } 804 805 /** 806 * @return the set of the regions contained in the table 807 */ 808 private List<RegionInfo> getTableRegions() throws IOException { 809 LOG.debug("get table regions: " + tableDir); 810 FileStatus[] regionDirs = FSUtils.listStatus(fs, tableDir, new FSUtils.RegionDirFilter(fs)); 811 if (regionDirs == null) return null; 812 813 List<RegionInfo> regions = new ArrayList<>(regionDirs.length); 814 for (int i = 0; i < regionDirs.length; ++i) { 815 RegionInfo hri = HRegionFileSystem.loadRegionInfoFileContent(fs, regionDirs[i].getPath()); 816 regions.add(hri); 817 } 818 LOG.debug("found " + regions.size() + " regions for table=" + 819 tableDesc.getTableName().getNameAsString()); 820 return regions; 821 } 822 823 /** 824 * Copy the snapshot files for a snapshot scanner, discards meta changes. 825 * @param conf 826 * @param fs 827 * @param rootDir 828 * @param restoreDir 829 * @param snapshotName 830 * @throws IOException 831 */ 832 public static RestoreMetaChanges copySnapshotForScanner(Configuration conf, FileSystem fs, 833 Path rootDir, Path restoreDir, String snapshotName) throws IOException { 834 // ensure that restore dir is not under root dir 835 if (!restoreDir.getFileSystem(conf).getUri().equals(rootDir.getFileSystem(conf).getUri())) { 836 throw new IllegalArgumentException("Filesystems for restore directory and HBase root " + 837 "directory should be the same"); 838 } 839 if (restoreDir.toUri().getPath().startsWith(rootDir.toUri().getPath() +"/")) { 840 throw new IllegalArgumentException("Restore directory cannot be a sub directory of HBase " + 841 "root directory. RootDir: " + rootDir + ", restoreDir: " + restoreDir); 842 } 843 844 Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotName, rootDir); 845 SnapshotDescription snapshotDesc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir); 846 SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc); 847 848 MonitoredTask status = TaskMonitor.get().createStatus( 849 "Restoring snapshot '" + snapshotName + "' to directory " + restoreDir); 850 ForeignExceptionDispatcher monitor = new ForeignExceptionDispatcher(); 851 852 // we send createBackRefs=false so that restored hfiles do not create back reference links 853 // in the base hbase root dir. 854 RestoreSnapshotHelper helper = new RestoreSnapshotHelper(conf, fs, 855 manifest, manifest.getTableDescriptor(), restoreDir, monitor, status, false); 856 RestoreMetaChanges metaChanges = helper.restoreHdfsRegions(); // TODO: parallelize. 857 858 if (LOG.isDebugEnabled()) { 859 LOG.debug("Restored table dir:" + restoreDir); 860 FSUtils.logFileSystemState(fs, restoreDir, LOG); 861 } 862 return metaChanges; 863 } 864 865 public static void restoreSnapshotAcl(SnapshotDescription snapshot, TableName newTableName, 866 Configuration conf) throws IOException { 867 if (snapshot.hasUsersAndPermissions() && snapshot.getUsersAndPermissions() != null) { 868 LOG.info("Restore snapshot acl to table. snapshot: " + snapshot + ", table: " + newTableName); 869 ListMultimap<String, Permission> perms = 870 ShadedAccessControlUtil.toUserTablePermissions(snapshot.getUsersAndPermissions()); 871 try (Connection conn = ConnectionFactory.createConnection(conf)) { 872 for (Entry<String, Permission> e : perms.entries()) { 873 String user = e.getKey(); 874 TablePermission tablePerm = (TablePermission) e.getValue(); 875 AccessControlClient.grant(conn, newTableName, user, tablePerm.getFamily(), 876 tablePerm.getQualifier(), tablePerm.getActions()); 877 } 878 } catch (Throwable e) { 879 throw new IOException("Grant acl into newly creatd table failed. snapshot: " + snapshot 880 + ", table: " + newTableName, e); 881 } 882 } 883 } 884}