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.impl; 019 020import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.JOB_NAME_CONF_KEY; 021 022import java.io.IOException; 023import java.util.ArrayList; 024import java.util.HashMap; 025import java.util.List; 026import java.util.TreeSet; 027import org.apache.commons.lang3.StringUtils; 028import org.apache.hadoop.conf.Configuration; 029import org.apache.hadoop.fs.FileSystem; 030import org.apache.hadoop.fs.LocatedFileStatus; 031import org.apache.hadoop.fs.Path; 032import org.apache.hadoop.fs.RemoteIterator; 033import org.apache.hadoop.hbase.TableName; 034import org.apache.hadoop.hbase.backup.BackupType; 035import org.apache.hadoop.hbase.backup.HBackupFileSystem; 036import org.apache.hadoop.hbase.backup.RestoreRequest; 037import org.apache.hadoop.hbase.backup.impl.BackupManifest.BackupImage; 038import org.apache.hadoop.hbase.backup.util.BackupUtils; 039import org.apache.hadoop.hbase.backup.util.RestoreTool; 040import org.apache.hadoop.hbase.client.Admin; 041import org.apache.hadoop.hbase.client.Connection; 042import org.apache.hadoop.hbase.io.hfile.HFile; 043import org.apache.yetus.audience.InterfaceAudience; 044import org.slf4j.Logger; 045import org.slf4j.LoggerFactory; 046 047/** 048 * Restore table implementation 049 */ 050@InterfaceAudience.Private 051public class RestoreTablesClient { 052 private static final Logger LOG = LoggerFactory.getLogger(RestoreTablesClient.class); 053 054 private Configuration conf; 055 private Connection conn; 056 private String backupId; 057 private TableName[] sTableArray; 058 private TableName[] tTableArray; 059 private String backupRootDir; 060 private Path restoreRootDir; 061 private boolean isOverwrite; 062 063 public RestoreTablesClient(Connection conn, RestoreRequest request) throws IOException { 064 this.backupRootDir = request.getBackupRootDir(); 065 this.backupId = request.getBackupId(); 066 this.sTableArray = request.getFromTables(); 067 this.tTableArray = request.getToTables(); 068 if (tTableArray == null || tTableArray.length == 0) { 069 this.tTableArray = sTableArray; 070 } 071 this.isOverwrite = request.isOverwrite(); 072 this.conn = conn; 073 this.conf = conn.getConfiguration(); 074 if (request.getRestoreRootDir() != null) { 075 restoreRootDir = new Path(request.getRestoreRootDir()); 076 } else { 077 FileSystem fs = FileSystem.get(conf); 078 this.restoreRootDir = BackupUtils.getTmpRestoreOutputDir(fs, conf); 079 } 080 } 081 082 /** 083 * Validate target tables. 084 * @param tTableArray target tables 085 * @param isOverwrite overwrite existing table 086 * @throws IOException exception 087 */ 088 private void checkTargetTables(TableName[] tTableArray, boolean isOverwrite) throws IOException { 089 ArrayList<TableName> existTableList = new ArrayList<>(); 090 ArrayList<TableName> disabledTableList = new ArrayList<>(); 091 092 // check if the tables already exist 093 try (Admin admin = conn.getAdmin()) { 094 for (TableName tableName : tTableArray) { 095 if (admin.tableExists(tableName)) { 096 existTableList.add(tableName); 097 if (admin.isTableDisabled(tableName)) { 098 disabledTableList.add(tableName); 099 } 100 } else { 101 LOG.info("HBase table " + tableName 102 + " does not exist. It will be created during restore process"); 103 } 104 } 105 } 106 107 if (existTableList.size() > 0) { 108 if (!isOverwrite) { 109 LOG.error("Existing table (" + existTableList + ") found in the restore target, please add " 110 + "\"-o\" as overwrite option in the command if you mean" 111 + " to restore to these existing tables"); 112 throw new IOException( 113 "Existing table found in target while no \"-o\" " + "as overwrite option found"); 114 } else { 115 if (disabledTableList.size() > 0) { 116 LOG.error("Found offline table in the restore target, " 117 + "please enable them before restore with \"-overwrite\" option"); 118 LOG.info("Offline table list in restore target: " + disabledTableList); 119 throw new IOException( 120 "Found offline table in the target when restore with \"-overwrite\" option"); 121 } 122 } 123 } 124 } 125 126 /** 127 * Restore operation handle each backupImage in array. 128 * @param images array BackupImage 129 * @param sTable table to be restored 130 * @param tTable table to be restored to 131 * @param truncateIfExists truncate table 132 * @throws IOException exception 133 */ 134 135 private void restoreImages(BackupImage[] images, TableName sTable, TableName tTable, 136 boolean truncateIfExists) throws IOException { 137 // First image MUST be image of a FULL backup 138 BackupImage image = images[0]; 139 String rootDir = image.getRootDir(); 140 String backupId = image.getBackupId(); 141 Path backupRoot = new Path(rootDir); 142 RestoreTool restoreTool = new RestoreTool(conf, backupRoot, restoreRootDir, backupId); 143 Path tableBackupPath = HBackupFileSystem.getTableBackupPath(sTable, backupRoot, backupId); 144 String lastIncrBackupId = images.length == 1 ? null : images[images.length - 1].getBackupId(); 145 // We need hFS only for full restore (see the code) 146 BackupManifest manifest = HBackupFileSystem.getManifest(conf, backupRoot, backupId); 147 if (manifest.getType() == BackupType.FULL) { 148 LOG.info("Restoring '" + sTable + "' to '" + tTable + "' from full" + " backup image " 149 + tableBackupPath.toString()); 150 conf.set(JOB_NAME_CONF_KEY, "Full_Restore-" + backupId + "-" + tTable); 151 restoreTool.fullRestoreTable(conn, tableBackupPath, sTable, tTable, truncateIfExists, 152 lastIncrBackupId); 153 conf.unset(JOB_NAME_CONF_KEY); 154 } else { // incremental Backup 155 throw new IOException("Unexpected backup type " + image.getType()); 156 } 157 158 if (images.length == 1) { 159 // full backup restore done 160 return; 161 } 162 163 List<Path> dirList = new ArrayList<>(); 164 // add full backup path 165 // full backup path comes first 166 for (int i = 1; i < images.length; i++) { 167 BackupImage im = images[i]; 168 String fileBackupDir = 169 HBackupFileSystem.getTableBackupDir(im.getRootDir(), im.getBackupId(), sTable); 170 List<Path> list = getFilesRecursively(fileBackupDir); 171 dirList.addAll(list); 172 173 } 174 175 if (dirList.isEmpty()) { 176 LOG.info("No incremental changes since full backup for '" + sTable 177 + "', skipping incremental restore step."); 178 return; 179 } 180 181 String dirs = StringUtils.join(dirList, ","); 182 LOG.info("Restoring '" + sTable + "' to '" + tTable + "' from log dirs: " + dirs); 183 Path[] paths = new Path[dirList.size()]; 184 dirList.toArray(paths); 185 conf.set(JOB_NAME_CONF_KEY, "Incremental_Restore-" + backupId + "-" + tTable); 186 restoreTool.incrementalRestoreTable(conn, tableBackupPath, paths, new TableName[] { sTable }, 187 new TableName[] { tTable }, lastIncrBackupId); 188 LOG.info(sTable + " has been successfully restored to " + tTable); 189 } 190 191 private List<Path> getFilesRecursively(String fileBackupDir) 192 throws IllegalArgumentException, IOException { 193 FileSystem fs = FileSystem.get(new Path(fileBackupDir).toUri(), new Configuration()); 194 List<Path> list = new ArrayList<>(); 195 RemoteIterator<LocatedFileStatus> it = fs.listFiles(new Path(fileBackupDir), true); 196 while (it.hasNext()) { 197 Path p = it.next().getPath(); 198 if (HFile.isHFileFormat(fs, p)) { 199 list.add(p); 200 } 201 } 202 return list; 203 } 204 205 /** 206 * Restore operation. Stage 2: resolved Backup Image dependency 207 * @param backupManifestMap : tableName, Manifest 208 * @param sTableArray The array of tables to be restored 209 * @param tTableArray The array of mapping tables to restore to 210 * @throws IOException exception 211 */ 212 private void restore(HashMap<TableName, BackupManifest> backupManifestMap, 213 TableName[] sTableArray, TableName[] tTableArray, boolean isOverwrite) throws IOException { 214 TreeSet<BackupImage> restoreImageSet = new TreeSet<>(); 215 216 for (int i = 0; i < sTableArray.length; i++) { 217 TableName table = sTableArray[i]; 218 219 BackupManifest manifest = backupManifestMap.get(table); 220 // Get the image list of this backup for restore in time order from old 221 // to new. 222 List<BackupImage> list = new ArrayList<>(); 223 list.add(manifest.getBackupImage()); 224 TreeSet<BackupImage> set = new TreeSet<>(list); 225 List<BackupImage> depList = manifest.getDependentListByTable(table); 226 set.addAll(depList); 227 BackupImage[] arr = new BackupImage[set.size()]; 228 set.toArray(arr); 229 restoreImages(arr, table, tTableArray[i], isOverwrite); 230 restoreImageSet.addAll(list); 231 if (restoreImageSet != null && !restoreImageSet.isEmpty()) { 232 LOG.info("Restore includes the following image(s):"); 233 for (BackupImage image : restoreImageSet) { 234 LOG.info("Backup: " + image.getBackupId() + " " 235 + HBackupFileSystem.getTableBackupDir(image.getRootDir(), image.getBackupId(), table)); 236 } 237 } 238 } 239 LOG.debug("restoreStage finished"); 240 } 241 242 static long getTsFromBackupId(String backupId) { 243 if (backupId == null) { 244 return 0; 245 } 246 return Long.parseLong(backupId.substring(backupId.lastIndexOf("_") + 1)); 247 } 248 249 static boolean withinRange(long a, long lower, long upper) { 250 return a >= lower && a <= upper; 251 } 252 253 public void execute() throws IOException { 254 // case VALIDATION: 255 // check the target tables 256 checkTargetTables(tTableArray, isOverwrite); 257 258 // case RESTORE_IMAGES: 259 HashMap<TableName, BackupManifest> backupManifestMap = new HashMap<>(); 260 // check and load backup image manifest for the tables 261 Path rootPath = new Path(backupRootDir); 262 HBackupFileSystem.checkImageManifestExist(backupManifestMap, sTableArray, conf, rootPath, 263 backupId); 264 265 restore(backupManifestMap, sTableArray, tTableArray, isOverwrite); 266 } 267}