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.backup.util; 019 020import java.io.FileNotFoundException; 021import java.io.IOException; 022import java.net.URLDecoder; 023import java.util.ArrayList; 024import java.util.Collections; 025import java.util.Comparator; 026import java.util.HashMap; 027import java.util.Iterator; 028import java.util.List; 029import java.util.Map; 030import java.util.Map.Entry; 031import java.util.TreeMap; 032import java.util.TreeSet; 033import org.apache.hadoop.conf.Configuration; 034import org.apache.hadoop.fs.FSDataOutputStream; 035import org.apache.hadoop.fs.FileStatus; 036import org.apache.hadoop.fs.FileSystem; 037import org.apache.hadoop.fs.LocatedFileStatus; 038import org.apache.hadoop.fs.Path; 039import org.apache.hadoop.fs.PathFilter; 040import org.apache.hadoop.fs.RemoteIterator; 041import org.apache.hadoop.fs.permission.FsPermission; 042import org.apache.hadoop.hbase.HConstants; 043import org.apache.hadoop.hbase.MetaTableAccessor; 044import org.apache.hadoop.hbase.ServerName; 045import org.apache.hadoop.hbase.TableName; 046import org.apache.hadoop.hbase.backup.BackupInfo; 047import org.apache.hadoop.hbase.backup.BackupRestoreConstants; 048import org.apache.hadoop.hbase.backup.HBackupFileSystem; 049import org.apache.hadoop.hbase.backup.RestoreRequest; 050import org.apache.hadoop.hbase.backup.impl.BackupManifest; 051import org.apache.hadoop.hbase.backup.impl.BackupManifest.BackupImage; 052import org.apache.hadoop.hbase.client.Admin; 053import org.apache.hadoop.hbase.client.Connection; 054import org.apache.hadoop.hbase.client.RegionInfo; 055import org.apache.hadoop.hbase.client.TableDescriptor; 056import org.apache.hadoop.hbase.master.region.MasterRegionFactory; 057import org.apache.hadoop.hbase.tool.BulkLoadHFiles; 058import org.apache.hadoop.hbase.util.CommonFSUtils; 059import org.apache.hadoop.hbase.util.EnvironmentEdgeManager; 060import org.apache.hadoop.hbase.util.FSTableDescriptors; 061import org.apache.hadoop.hbase.util.FSUtils; 062import org.apache.hadoop.hbase.wal.AbstractFSWALProvider; 063import org.apache.yetus.audience.InterfaceAudience; 064import org.slf4j.Logger; 065import org.slf4j.LoggerFactory; 066 067import org.apache.hbase.thirdparty.com.google.common.base.Splitter; 068import org.apache.hbase.thirdparty.com.google.common.collect.Iterables; 069import org.apache.hbase.thirdparty.com.google.common.collect.Iterators; 070 071/** 072 * A collection for methods used by multiple classes to backup HBase tables. 073 */ 074@InterfaceAudience.Private 075public final class BackupUtils { 076 private static final Logger LOG = LoggerFactory.getLogger(BackupUtils.class); 077 public static final String LOGNAME_SEPARATOR = "."; 078 public static final int MILLISEC_IN_HOUR = 3600000; 079 080 private BackupUtils() { 081 throw new AssertionError("Instantiating utility class..."); 082 } 083 084 /** 085 * Loop through the RS log timestamp map for the tables, for each RS, find the min timestamp value 086 * for the RS among the tables. 087 * @param rsLogTimestampMap timestamp map 088 * @return the min timestamp of each RS 089 */ 090 public static Map<String, Long> 091 getRSLogTimestampMins(Map<TableName, Map<String, Long>> rsLogTimestampMap) { 092 if (rsLogTimestampMap == null || rsLogTimestampMap.isEmpty()) { 093 return null; 094 } 095 096 HashMap<String, Long> rsLogTimestampMins = new HashMap<>(); 097 HashMap<String, HashMap<TableName, Long>> rsLogTimestampMapByRS = new HashMap<>(); 098 099 for (Entry<TableName, Map<String, Long>> tableEntry : rsLogTimestampMap.entrySet()) { 100 TableName table = tableEntry.getKey(); 101 Map<String, Long> rsLogTimestamp = tableEntry.getValue(); 102 for (Entry<String, Long> rsEntry : rsLogTimestamp.entrySet()) { 103 String rs = rsEntry.getKey(); 104 Long ts = rsEntry.getValue(); 105 rsLogTimestampMapByRS.putIfAbsent(rs, new HashMap<>()); 106 rsLogTimestampMapByRS.get(rs).put(table, ts); 107 } 108 } 109 110 for (Entry<String, HashMap<TableName, Long>> entry : rsLogTimestampMapByRS.entrySet()) { 111 String rs = entry.getKey(); 112 rsLogTimestampMins.put(rs, BackupUtils.getMinValue(entry.getValue())); 113 } 114 115 return rsLogTimestampMins; 116 } 117 118 /** 119 * copy out Table RegionInfo into incremental backup image need to consider move this logic into 120 * HBackupFileSystem 121 * @param conn connection 122 * @param backupInfo backup info 123 * @param conf configuration 124 * @throws IOException exception 125 */ 126 public static void copyTableRegionInfo(Connection conn, BackupInfo backupInfo, Configuration conf) 127 throws IOException { 128 Path rootDir = CommonFSUtils.getRootDir(conf); 129 FileSystem fs = rootDir.getFileSystem(conf); 130 131 // for each table in the table set, copy out the table info and region 132 // info files in the correct directory structure 133 try (Admin admin = conn.getAdmin()) { 134 for (TableName table : backupInfo.getTables()) { 135 if (!admin.tableExists(table)) { 136 LOG.warn("Table " + table + " does not exists, skipping it."); 137 continue; 138 } 139 TableDescriptor orig = FSTableDescriptors.getTableDescriptorFromFs(fs, rootDir, table); 140 141 // write a copy of descriptor to the target directory 142 Path target = new Path(backupInfo.getTableBackupDir(table)); 143 FileSystem targetFs = target.getFileSystem(conf); 144 FSTableDescriptors descriptors = 145 new FSTableDescriptors(targetFs, CommonFSUtils.getRootDir(conf)); 146 descriptors.createTableDescriptorForTableDirectory(target, orig, false); 147 LOG.debug("Attempting to copy table info for:" + table + " target: " + target 148 + " descriptor: " + orig); 149 LOG.debug("Finished copying tableinfo."); 150 List<RegionInfo> regions = MetaTableAccessor.getTableRegions(conn, table); 151 // For each region, write the region info to disk 152 LOG.debug("Starting to write region info for table " + table); 153 for (RegionInfo regionInfo : regions) { 154 Path regionDir = FSUtils 155 .getRegionDirFromTableDir(new Path(backupInfo.getTableBackupDir(table)), regionInfo); 156 regionDir = new Path(backupInfo.getTableBackupDir(table), regionDir.getName()); 157 writeRegioninfoOnFilesystem(conf, targetFs, regionDir, regionInfo); 158 } 159 LOG.debug("Finished writing region info for table " + table); 160 } 161 } 162 } 163 164 /** 165 * Write the .regioninfo file on-disk. 166 */ 167 public static void writeRegioninfoOnFilesystem(final Configuration conf, final FileSystem fs, 168 final Path regionInfoDir, RegionInfo regionInfo) throws IOException { 169 final byte[] content = RegionInfo.toDelimitedByteArray(regionInfo); 170 Path regionInfoFile = new Path(regionInfoDir, "." + HConstants.REGIONINFO_QUALIFIER_STR); 171 // First check to get the permissions 172 FsPermission perms = CommonFSUtils.getFilePermissions(fs, conf, HConstants.DATA_FILE_UMASK_KEY); 173 // Write the RegionInfo file content 174 FSDataOutputStream out = FSUtils.create(conf, fs, regionInfoFile, perms, null); 175 try { 176 out.write(content); 177 } finally { 178 out.close(); 179 } 180 } 181 182 /** 183 * Parses hostname:port from WAL file path 184 * @param p path to WAL file 185 * @return hostname:port 186 */ 187 public static String parseHostNameFromLogFile(Path p) { 188 try { 189 if (AbstractFSWALProvider.isArchivedLogFile(p)) { 190 return BackupUtils.parseHostFromOldLog(p); 191 } else { 192 ServerName sname = AbstractFSWALProvider.getServerNameFromWALDirectoryName(p); 193 if (sname != null) { 194 return sname.getAddress().toString(); 195 } else { 196 LOG.error("Skip log file (can't parse): " + p); 197 return null; 198 } 199 } 200 } catch (Exception e) { 201 LOG.error("Skip log file (can't parse): " + p, e); 202 return null; 203 } 204 } 205 206 /** 207 * Returns WAL file name 208 * @param walFileName WAL file name 209 * @return WAL file name 210 */ 211 public static String getUniqueWALFileNamePart(String walFileName) { 212 return getUniqueWALFileNamePart(new Path(walFileName)); 213 } 214 215 /** 216 * Returns WAL file name 217 * @param p WAL file path 218 * @return WAL file name 219 */ 220 public static String getUniqueWALFileNamePart(Path p) { 221 return p.getName(); 222 } 223 224 /** 225 * Get the total length of files under the given directory recursively. 226 * @param fs The hadoop file system 227 * @param dir The target directory 228 * @return the total length of files 229 * @throws IOException exception 230 */ 231 public static long getFilesLength(FileSystem fs, Path dir) throws IOException { 232 long totalLength = 0; 233 FileStatus[] files = CommonFSUtils.listStatus(fs, dir); 234 if (files != null) { 235 for (FileStatus fileStatus : files) { 236 if (fileStatus.isDirectory()) { 237 totalLength += getFilesLength(fs, fileStatus.getPath()); 238 } else { 239 totalLength += fileStatus.getLen(); 240 } 241 } 242 } 243 return totalLength; 244 } 245 246 /** 247 * Get list of all old WAL files (WALs and archive) 248 * @param c configuration 249 * @param hostTimestampMap {host,timestamp} map 250 * @return list of WAL files 251 * @throws IOException exception 252 */ 253 public static List<String> getWALFilesOlderThan(final Configuration c, 254 final HashMap<String, Long> hostTimestampMap) throws IOException { 255 Path walRootDir = CommonFSUtils.getWALRootDir(c); 256 Path logDir = new Path(walRootDir, HConstants.HREGION_LOGDIR_NAME); 257 Path oldLogDir = new Path(walRootDir, HConstants.HREGION_OLDLOGDIR_NAME); 258 List<String> logFiles = new ArrayList<>(); 259 260 PathFilter filter = p -> { 261 try { 262 if (AbstractFSWALProvider.isMetaFile(p)) { 263 return false; 264 } 265 String host = parseHostNameFromLogFile(p); 266 if (host == null) { 267 return false; 268 } 269 Long oldTimestamp = hostTimestampMap.get(host); 270 Long currentLogTS = BackupUtils.getCreationTime(p); 271 return currentLogTS <= oldTimestamp; 272 } catch (Exception e) { 273 LOG.warn("Can not parse" + p, e); 274 return false; 275 } 276 }; 277 FileSystem walFs = CommonFSUtils.getWALFileSystem(c); 278 logFiles = BackupUtils.getFiles(walFs, logDir, logFiles, filter); 279 logFiles = BackupUtils.getFiles(walFs, oldLogDir, logFiles, filter); 280 return logFiles; 281 } 282 283 public static TableName[] parseTableNames(String tables) { 284 if (tables == null) { 285 return null; 286 } 287 return Splitter.on(BackupRestoreConstants.TABLENAME_DELIMITER_IN_COMMAND).splitToStream(tables) 288 .map(TableName::valueOf).toArray(TableName[]::new); 289 } 290 291 /** 292 * Check whether the backup path exist 293 * @param backupStr backup 294 * @param conf configuration 295 * @return Yes if path exists 296 * @throws IOException exception 297 */ 298 public static boolean checkPathExist(String backupStr, Configuration conf) throws IOException { 299 boolean isExist = false; 300 Path backupPath = new Path(backupStr); 301 FileSystem fileSys = backupPath.getFileSystem(conf); 302 String targetFsScheme = fileSys.getUri().getScheme(); 303 if (LOG.isTraceEnabled()) { 304 LOG.trace("Schema of given url: " + backupStr + " is: " + targetFsScheme); 305 } 306 if (fileSys.exists(backupPath)) { 307 isExist = true; 308 } 309 return isExist; 310 } 311 312 /** 313 * Check target path first, confirm it doesn't exist before backup 314 * @param backupRootPath backup destination path 315 * @param conf configuration 316 * @throws IOException exception 317 */ 318 public static void checkTargetDir(String backupRootPath, Configuration conf) throws IOException { 319 boolean targetExists; 320 try { 321 targetExists = checkPathExist(backupRootPath, conf); 322 } catch (IOException e) { 323 String expMsg = e.getMessage(); 324 String newMsg = null; 325 if (expMsg.contains("No FileSystem for scheme")) { 326 newMsg = 327 "Unsupported filesystem scheme found in the backup target url. Error Message: " + expMsg; 328 LOG.error(newMsg); 329 throw new IOException(newMsg); 330 } else { 331 throw e; 332 } 333 } 334 335 if (targetExists) { 336 LOG.info("Using existing backup root dir: " + backupRootPath); 337 } else { 338 LOG.info("Backup root dir " + backupRootPath + " does not exist. Will be created."); 339 } 340 } 341 342 /** 343 * Get the min value for all the Values a map. 344 * @param map map 345 * @return the min value 346 */ 347 public static <T> Long getMinValue(Map<T, Long> map) { 348 Long minTimestamp = null; 349 if (map != null) { 350 ArrayList<Long> timestampList = new ArrayList<>(map.values()); 351 Collections.sort(timestampList); 352 // The min among all the RS log timestamps will be kept in backup system table table. 353 minTimestamp = timestampList.get(0); 354 } 355 return minTimestamp; 356 } 357 358 /** 359 * Parses host name:port from archived WAL path 360 * @param p path 361 * @return host name 362 */ 363 public static String parseHostFromOldLog(Path p) { 364 // Skip master wals 365 if (p.getName().endsWith(MasterRegionFactory.ARCHIVED_WAL_SUFFIX)) { 366 return null; 367 } 368 try { 369 String urlDecodedName = URLDecoder.decode(p.getName(), "UTF8"); 370 Iterable<String> nameSplitsOnComma = Splitter.on(",").split(urlDecodedName); 371 String host = Iterables.get(nameSplitsOnComma, 0); 372 String port = Iterables.get(nameSplitsOnComma, 1); 373 return host + ":" + port; 374 } catch (Exception e) { 375 LOG.warn("Skip log file (can't parse): {}", p); 376 return null; 377 } 378 } 379 380 /** 381 * Given the log file, parse the timestamp from the file name. The timestamp is the last number. 382 * @param p a path to the log file 383 * @return the timestamp 384 * @throws IOException exception 385 */ 386 public static Long getCreationTime(Path p) throws IOException { 387 int idx = p.getName().lastIndexOf(LOGNAME_SEPARATOR); 388 if (idx < 0) { 389 throw new IOException("Cannot parse timestamp from path " + p); 390 } 391 String ts = p.getName().substring(idx + 1); 392 return Long.parseLong(ts); 393 } 394 395 public static List<String> getFiles(FileSystem fs, Path rootDir, List<String> files, 396 PathFilter filter) throws IOException { 397 RemoteIterator<LocatedFileStatus> it = fs.listFiles(rootDir, true); 398 399 while (it.hasNext()) { 400 LocatedFileStatus lfs = it.next(); 401 if (lfs.isDirectory()) { 402 continue; 403 } 404 // apply filter 405 if (filter.accept(lfs.getPath())) { 406 files.add(lfs.getPath().toString()); 407 } 408 } 409 return files; 410 } 411 412 public static void cleanupBackupData(BackupInfo context, Configuration conf) throws IOException { 413 cleanupHLogDir(context, conf); 414 cleanupTargetDir(context, conf); 415 } 416 417 /** 418 * Clean up directories which are generated when DistCp copying hlogs 419 * @param backupInfo backup info 420 * @param conf configuration 421 * @throws IOException exception 422 */ 423 private static void cleanupHLogDir(BackupInfo backupInfo, Configuration conf) throws IOException { 424 String logDir = backupInfo.getHLogTargetDir(); 425 if (logDir == null) { 426 LOG.warn("No log directory specified for " + backupInfo.getBackupId()); 427 return; 428 } 429 430 Path rootPath = new Path(logDir).getParent(); 431 FileSystem fs = FileSystem.get(rootPath.toUri(), conf); 432 FileStatus[] files = listStatus(fs, rootPath, null); 433 if (files == null) { 434 return; 435 } 436 for (FileStatus file : files) { 437 LOG.debug("Delete log files: " + file.getPath().getName()); 438 fs.delete(file.getPath(), true); 439 } 440 } 441 442 private static void cleanupTargetDir(BackupInfo backupInfo, Configuration conf) { 443 try { 444 // clean up the data at target directory 445 LOG.debug("Trying to cleanup up target dir : " + backupInfo.getBackupId()); 446 String targetDir = backupInfo.getBackupRootDir(); 447 if (targetDir == null) { 448 LOG.warn("No target directory specified for " + backupInfo.getBackupId()); 449 return; 450 } 451 452 FileSystem outputFs = FileSystem.get(new Path(backupInfo.getBackupRootDir()).toUri(), conf); 453 454 for (TableName table : backupInfo.getTables()) { 455 Path targetDirPath = new Path( 456 getTableBackupDir(backupInfo.getBackupRootDir(), backupInfo.getBackupId(), table)); 457 if (outputFs.delete(targetDirPath, true)) { 458 LOG.info("Cleaning up backup data at " + targetDirPath.toString() + " done."); 459 } else { 460 LOG.info("No data has been found in " + targetDirPath.toString() + "."); 461 } 462 463 Path tableDir = targetDirPath.getParent(); 464 FileStatus[] backups = listStatus(outputFs, tableDir, null); 465 if (backups == null || backups.length == 0) { 466 outputFs.delete(tableDir, true); 467 LOG.debug(tableDir.toString() + " is empty, remove it."); 468 } 469 } 470 outputFs.delete(new Path(targetDir, backupInfo.getBackupId()), true); 471 } catch (IOException e1) { 472 LOG.error("Cleaning up backup data of " + backupInfo.getBackupId() + " at " 473 + backupInfo.getBackupRootDir() + " failed due to " + e1.getMessage() + "."); 474 } 475 } 476 477 /** 478 * Given the backup root dir, backup id and the table name, return the backup image location, 479 * which is also where the backup manifest file is. return value look like: 480 * "hdfs://backup.hbase.org:9000/user/biadmin/backup1/backup_1396650096738/default/t1_dn/" 481 * @param backupRootDir backup root directory 482 * @param backupId backup id 483 * @param tableName table name 484 * @return backupPath String for the particular table 485 */ 486 public static String getTableBackupDir(String backupRootDir, String backupId, 487 TableName tableName) { 488 return backupRootDir + Path.SEPARATOR + backupId + Path.SEPARATOR 489 + tableName.getNamespaceAsString() + Path.SEPARATOR + tableName.getQualifierAsString() 490 + Path.SEPARATOR; 491 } 492 493 /** 494 * Sort history list by start time in descending order. 495 * @param historyList history list 496 * @return sorted list of BackupCompleteData 497 */ 498 public static ArrayList<BackupInfo> sortHistoryListDesc(ArrayList<BackupInfo> historyList) { 499 ArrayList<BackupInfo> list = new ArrayList<>(); 500 TreeMap<String, BackupInfo> map = new TreeMap<>(); 501 for (BackupInfo h : historyList) { 502 map.put(Long.toString(h.getStartTs()), h); 503 } 504 Iterator<String> i = map.descendingKeySet().iterator(); 505 while (i.hasNext()) { 506 list.add(map.get(i.next())); 507 } 508 return list; 509 } 510 511 /** 512 * Calls fs.listStatus() and treats FileNotFoundException as non-fatal This accommodates 513 * differences between hadoop versions, where hadoop 1 does not throw a FileNotFoundException, and 514 * return an empty FileStatus[] while Hadoop 2 will throw FileNotFoundException. 515 * @param fs file system 516 * @param dir directory 517 * @param filter path filter 518 * @return null if dir is empty or doesn't exist, otherwise FileStatus array 519 */ 520 public static FileStatus[] listStatus(final FileSystem fs, final Path dir, 521 final PathFilter filter) throws IOException { 522 FileStatus[] status = null; 523 try { 524 status = filter == null ? fs.listStatus(dir) : fs.listStatus(dir, filter); 525 } catch (FileNotFoundException fnfe) { 526 // if directory doesn't exist, return null 527 if (LOG.isTraceEnabled()) { 528 LOG.trace(dir + " doesn't exist"); 529 } 530 } 531 532 if (status == null || status.length < 1) { 533 return null; 534 } 535 536 return status; 537 } 538 539 /** 540 * Return the 'path' component of a Path. In Hadoop, Path is a URI. This method returns the 'path' 541 * component of a Path's URI: e.g. If a Path is 542 * <code>hdfs://example.org:9000/hbase_trunk/TestTable/compaction.dir</code>, this method returns 543 * <code>/hbase_trunk/TestTable/compaction.dir</code>. This method is useful if you want to print 544 * out a Path without qualifying Filesystem instance. 545 * @param p file system Path whose 'path' component we are to return. 546 * @return Path portion of the Filesystem 547 */ 548 public static String getPath(Path p) { 549 return p.toUri().getPath(); 550 } 551 552 /** 553 * Given the backup root dir and the backup id, return the log file location for an incremental 554 * backup. 555 * @param backupRootDir backup root directory 556 * @param backupId backup id 557 * @return logBackupDir: ".../user/biadmin/backup1/WALs/backup_1396650096738" 558 */ 559 public static String getLogBackupDir(String backupRootDir, String backupId) { 560 return backupRootDir + Path.SEPARATOR + backupId + Path.SEPARATOR 561 + HConstants.HREGION_LOGDIR_NAME; 562 } 563 564 private static List<BackupInfo> getHistory(Configuration conf, Path backupRootPath) 565 throws IOException { 566 // Get all (n) history from backup root destination 567 568 FileSystem fs = FileSystem.get(backupRootPath.toUri(), conf); 569 RemoteIterator<LocatedFileStatus> it = fs.listLocatedStatus(backupRootPath); 570 571 List<BackupInfo> infos = new ArrayList<>(); 572 while (it.hasNext()) { 573 LocatedFileStatus lfs = it.next(); 574 575 if (!lfs.isDirectory()) { 576 continue; 577 } 578 579 String backupId = lfs.getPath().getName(); 580 try { 581 BackupInfo info = loadBackupInfo(backupRootPath, backupId, fs); 582 infos.add(info); 583 } catch (IOException e) { 584 LOG.error("Can not load backup info from: " + lfs.getPath(), e); 585 } 586 } 587 // Sort 588 Collections.sort(infos, new Comparator<BackupInfo>() { 589 @Override 590 public int compare(BackupInfo o1, BackupInfo o2) { 591 long ts1 = getTimestamp(o1.getBackupId()); 592 long ts2 = getTimestamp(o2.getBackupId()); 593 594 if (ts1 == ts2) { 595 return 0; 596 } 597 598 return ts1 < ts2 ? 1 : -1; 599 } 600 601 private long getTimestamp(String backupId) { 602 return Long.parseLong(Iterators.get(Splitter.on('_').split(backupId).iterator(), 1)); 603 } 604 }); 605 return infos; 606 } 607 608 public static List<BackupInfo> getHistory(Configuration conf, int n, Path backupRootPath, 609 BackupInfo.Filter... filters) throws IOException { 610 List<BackupInfo> infos = getHistory(conf, backupRootPath); 611 List<BackupInfo> ret = new ArrayList<>(); 612 for (BackupInfo info : infos) { 613 if (ret.size() == n) { 614 break; 615 } 616 boolean passed = true; 617 for (int i = 0; i < filters.length; i++) { 618 if (!filters[i].apply(info)) { 619 passed = false; 620 break; 621 } 622 } 623 if (passed) { 624 ret.add(info); 625 } 626 } 627 return ret; 628 } 629 630 public static BackupInfo loadBackupInfo(Path backupRootPath, String backupId, FileSystem fs) 631 throws IOException { 632 Path backupPath = new Path(backupRootPath, backupId); 633 634 RemoteIterator<LocatedFileStatus> it = fs.listFiles(backupPath, true); 635 while (it.hasNext()) { 636 LocatedFileStatus lfs = it.next(); 637 if (lfs.getPath().getName().equals(BackupManifest.MANIFEST_FILE_NAME)) { 638 // Load BackupManifest 639 BackupManifest manifest = new BackupManifest(fs, lfs.getPath().getParent()); 640 BackupInfo info = manifest.toBackupInfo(); 641 return info; 642 } 643 } 644 return null; 645 } 646 647 /** 648 * Create restore request. 649 * @param backupRootDir backup root dir 650 * @param backupId backup id 651 * @param check check only 652 * @param fromTables table list from 653 * @param toTables table list to 654 * @param isOverwrite overwrite data 655 * @return request obkect 656 */ 657 public static RestoreRequest createRestoreRequest(String backupRootDir, String backupId, 658 boolean check, TableName[] fromTables, TableName[] toTables, boolean isOverwrite) { 659 RestoreRequest.Builder builder = new RestoreRequest.Builder(); 660 RestoreRequest request = 661 builder.withBackupRootDir(backupRootDir).withBackupId(backupId).withCheck(check) 662 .withFromTables(fromTables).withToTables(toTables).withOvewrite(isOverwrite).build(); 663 return request; 664 } 665 666 public static boolean validate(HashMap<TableName, BackupManifest> backupManifestMap, 667 Configuration conf) throws IOException { 668 boolean isValid = true; 669 670 for (Entry<TableName, BackupManifest> manifestEntry : backupManifestMap.entrySet()) { 671 TableName table = manifestEntry.getKey(); 672 TreeSet<BackupImage> imageSet = new TreeSet<>(); 673 674 ArrayList<BackupImage> depList = manifestEntry.getValue().getDependentListByTable(table); 675 if (depList != null && !depList.isEmpty()) { 676 imageSet.addAll(depList); 677 } 678 679 LOG.info("Dependent image(s) from old to new:"); 680 for (BackupImage image : imageSet) { 681 String imageDir = 682 HBackupFileSystem.getTableBackupDir(image.getRootDir(), image.getBackupId(), table); 683 if (!BackupUtils.checkPathExist(imageDir, conf)) { 684 LOG.error("ERROR: backup image does not exist: " + imageDir); 685 isValid = false; 686 break; 687 } 688 LOG.info("Backup image: " + image.getBackupId() + " for '" + table + "' is available"); 689 } 690 } 691 return isValid; 692 } 693 694 public static Path getBulkOutputDir(Path restoreRootDir, String tableName, Configuration conf, 695 boolean deleteOnExit) throws IOException { 696 FileSystem fs = restoreRootDir.getFileSystem(conf); 697 Path path = new Path(restoreRootDir, 698 "bulk_output-" + tableName + "-" + EnvironmentEdgeManager.currentTime()); 699 if (deleteOnExit) { 700 fs.deleteOnExit(path); 701 } 702 return path; 703 } 704 705 public static Path getBulkOutputDir(Path restoreRootDir, String tableName, Configuration conf) 706 throws IOException { 707 return getBulkOutputDir(restoreRootDir, tableName, conf, true); 708 } 709 710 public static Path getBulkOutputDir(String tableName, Configuration conf, boolean deleteOnExit) 711 throws IOException { 712 FileSystem fs = FileSystem.get(conf); 713 return getBulkOutputDir(getTmpRestoreOutputDir(fs, conf), tableName, conf, deleteOnExit); 714 } 715 716 /** 717 * Build temporary output path 718 * @param fs filesystem for default output dir 719 * @param conf configuration 720 * @return output path 721 */ 722 public static Path getTmpRestoreOutputDir(FileSystem fs, Configuration conf) { 723 String tmp = 724 conf.get(HConstants.TEMPORARY_FS_DIRECTORY_KEY, fs.getHomeDirectory() + "/hbase-staging"); 725 return new Path(tmp); 726 } 727 728 public static String getFileNameCompatibleString(TableName table) { 729 return table.getNamespaceAsString() + "-" + table.getQualifierAsString(); 730 } 731 732 public static boolean failed(int result) { 733 return result != 0; 734 } 735 736 public static boolean succeeded(int result) { 737 return result == 0; 738 } 739 740 public static BulkLoadHFiles createLoader(Configuration config) { 741 // set configuration for restore: 742 // LoadIncrementalHFile needs more time 743 // <name>hbase.rpc.timeout</name> <value>600000</value> 744 // calculates 745 Configuration conf = new Configuration(config); 746 conf.setInt(HConstants.HBASE_RPC_TIMEOUT_KEY, MILLISEC_IN_HOUR); 747 748 // By default, it is 32 and loader will fail if # of files in any region exceed this 749 // limit. Bad for snapshot restore. 750 conf.setInt(BulkLoadHFiles.MAX_FILES_PER_REGION_PER_FAMILY, Integer.MAX_VALUE); 751 conf.set(BulkLoadHFiles.IGNORE_UNMATCHED_CF_CONF_KEY, "yes"); 752 return BulkLoadHFiles.create(conf); 753 } 754 755 public static String findMostRecentBackupId(String[] backupIds) { 756 long recentTimestamp = Long.MIN_VALUE; 757 for (String backupId : backupIds) { 758 long ts = Long.parseLong(Iterators.get(Splitter.on('_').split(backupId).iterator(), 1)); 759 if (ts > recentTimestamp) { 760 recentTimestamp = ts; 761 } 762 } 763 return BackupRestoreConstants.BACKUPID_PREFIX + recentTimestamp; 764 } 765 766}