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.OPTION_BACKUP_LIST_DESC;
021import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_BANDWIDTH;
022import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_BANDWIDTH_DESC;
023import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_DEBUG;
024import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_DEBUG_DESC;
025import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_IGNORECHECKSUM;
026import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_IGNORECHECKSUM_DESC;
027import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_KEEP;
028import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_KEEP_DESC;
029import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_LIST;
030import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_PATH;
031import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_PATH_DESC;
032import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_RECORD_NUMBER;
033import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_RECORD_NUMBER_DESC;
034import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_SET;
035import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_SET_BACKUP_DESC;
036import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_SET_DESC;
037import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_TABLE;
038import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_TABLE_DESC;
039import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_TABLE_LIST_DESC;
040import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_WORKERS;
041import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_WORKERS_DESC;
042import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_YARN_QUEUE_NAME;
043import static org.apache.hadoop.hbase.backup.BackupRestoreConstants.OPTION_YARN_QUEUE_NAME_DESC;
044
045import java.io.IOException;
046import java.net.URI;
047import java.util.List;
048import org.apache.commons.lang3.StringUtils;
049import org.apache.hadoop.conf.Configuration;
050import org.apache.hadoop.conf.Configured;
051import org.apache.hadoop.fs.FileSystem;
052import org.apache.hadoop.fs.Path;
053import org.apache.hadoop.hbase.HBaseConfiguration;
054import org.apache.hadoop.hbase.TableName;
055import org.apache.hadoop.hbase.backup.BackupAdmin;
056import org.apache.hadoop.hbase.backup.BackupInfo;
057import org.apache.hadoop.hbase.backup.BackupInfo.BackupState;
058import org.apache.hadoop.hbase.backup.BackupRequest;
059import org.apache.hadoop.hbase.backup.BackupRestoreConstants;
060import org.apache.hadoop.hbase.backup.BackupRestoreConstants.BackupCommand;
061import org.apache.hadoop.hbase.backup.BackupType;
062import org.apache.hadoop.hbase.backup.HBackupFileSystem;
063import org.apache.hadoop.hbase.backup.util.BackupSet;
064import org.apache.hadoop.hbase.backup.util.BackupUtils;
065import org.apache.hadoop.hbase.client.Connection;
066import org.apache.hadoop.hbase.client.ConnectionFactory;
067import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
068import org.apache.yetus.audience.InterfaceAudience;
069
070import org.apache.hbase.thirdparty.com.google.common.base.Splitter;
071import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
072import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLine;
073import org.apache.hbase.thirdparty.org.apache.commons.cli.HelpFormatter;
074import org.apache.hbase.thirdparty.org.apache.commons.cli.Options;
075
076/**
077 * General backup commands, options and usage messages
078 */
079@InterfaceAudience.Private
080public final class BackupCommands {
081  public final static String INCORRECT_USAGE = "Incorrect usage";
082
083  public final static String TOP_LEVEL_NOT_ALLOWED =
084    "Top level (root) folder is not allowed to be a backup destination";
085
086  public static final String USAGE = "Usage: hbase backup COMMAND [command-specific arguments]\n"
087    + "where COMMAND is one of:\n" + "  create     create a new backup image\n"
088    + "  delete     delete an existing backup image\n"
089    + "  describe   show the detailed information of a backup image\n"
090    + "  history    show history of all successful backups\n"
091    + "  progress   show the progress of the latest backup request\n"
092    + "  set        backup set management\n" + "  repair     repair backup system table\n"
093    + "  merge      merge backup images\n"
094    + "Run \'hbase backup COMMAND -h\' to see help message for each command\n";
095
096  public static final String CREATE_CMD_USAGE =
097    "Usage: hbase backup create <type> <backup_path> [options]\n"
098      + "  type           \"full\" to create a full backup image\n"
099      + "                 \"incremental\" to create an incremental backup image\n"
100      + "  backup_path     Full path to store the backup image\n";
101
102  public static final String PROGRESS_CMD_USAGE = "Usage: hbase backup progress <backup_id>\n"
103    + "  backup_id       Backup image id (optional). If no id specified, the command will show\n"
104    + "                  progress for currently running backup session.";
105  public static final String NO_INFO_FOUND = "No info was found for backup id: ";
106  public static final String NO_ACTIVE_SESSION_FOUND = "No active backup sessions found.";
107
108  public static final String DESCRIBE_CMD_USAGE =
109    "Usage: hbase backup describe <backup_id>\n" + "  backup_id       Backup image id\n";
110
111  public static final String HISTORY_CMD_USAGE = "Usage: hbase backup history [options]";
112
113  public static final String DELETE_CMD_USAGE = "Usage: hbase backup delete [options]";
114
115  public static final String REPAIR_CMD_USAGE = "Usage: hbase backup repair\n";
116
117  public static final String SET_CMD_USAGE = "Usage: hbase backup set COMMAND [name] [tables]\n"
118    + "  name            Backup set name\n" + "  tables          Comma separated list of tables.\n"
119    + "COMMAND is one of:\n" + "  add             add tables to a set, create a set if needed\n"
120    + "  remove          remove tables from a set\n"
121    + "  list            list all backup sets in the system\n" + "  describe        describe set\n"
122    + "  delete          delete backup set\n";
123  public static final String MERGE_CMD_USAGE = "Usage: hbase backup merge [backup_ids]\n"
124    + "  backup_ids      Comma separated list of backup image ids.\n";
125
126  public static final String USAGE_FOOTER = "";
127
128  public static abstract class Command extends Configured {
129    CommandLine cmdline;
130    Connection conn;
131
132    Command(Configuration conf) {
133      if (conf == null) {
134        conf = HBaseConfiguration.create();
135      }
136      setConf(conf);
137    }
138
139    public void execute() throws IOException {
140      if (cmdline.hasOption("h") || cmdline.hasOption("help")) {
141        printUsage();
142        throw new IOException(INCORRECT_USAGE);
143      }
144
145      if (cmdline.hasOption(OPTION_YARN_QUEUE_NAME)) {
146        String queueName = cmdline.getOptionValue(OPTION_YARN_QUEUE_NAME);
147        // Set MR job queuename to configuration
148        getConf().set("mapreduce.job.queuename", queueName);
149      }
150
151      // Create connection
152      conn = ConnectionFactory.createConnection(getConf());
153      if (requiresNoActiveSession()) {
154        // Check active session
155        try (BackupSystemTable table = new BackupSystemTable(conn)) {
156          List<BackupInfo> sessions = table.getBackupInfos(BackupState.RUNNING);
157
158          if (sessions.size() > 0) {
159            System.err.println("Found backup session in a RUNNING state: ");
160            System.err.println(sessions.get(0));
161            System.err.println("This may indicate that a previous session has failed abnormally.");
162            System.err.println("In this case, backup recovery is recommended.");
163            throw new IOException("Active session found, aborted command execution");
164          }
165        }
166      }
167      if (requiresConsistentState()) {
168        // Check failed delete
169        try (BackupSystemTable table = new BackupSystemTable(conn)) {
170          String[] ids = table.getListOfBackupIdsFromDeleteOperation();
171
172          if (ids != null && ids.length > 0) {
173            System.err.println("Found failed backup DELETE coommand. ");
174            System.err.println("Backup system recovery is required.");
175            throw new IOException("Failed backup DELETE found, aborted command execution");
176          }
177
178          ids = table.getListOfBackupIdsFromMergeOperation();
179          if (ids != null && ids.length > 0) {
180            System.err.println("Found failed backup MERGE coommand. ");
181            System.err.println("Backup system recovery is required.");
182            throw new IOException("Failed backup MERGE found, aborted command execution");
183          }
184        }
185      }
186    }
187
188    public void finish() throws IOException {
189      if (conn != null) {
190        conn.close();
191      }
192    }
193
194    protected abstract void printUsage();
195
196    /**
197     * The command can't be run if active backup session is in progress
198     * @return true if no active sessions are in progress
199     */
200    protected boolean requiresNoActiveSession() {
201      return false;
202    }
203
204    /**
205     * Command requires consistent state of a backup system Backup system may become inconsistent
206     * because of an abnormal termination of a backup session or delete command
207     * @return true, if yes
208     */
209    protected boolean requiresConsistentState() {
210      return false;
211    }
212  }
213
214  private BackupCommands() {
215    throw new AssertionError("Instantiating utility class...");
216  }
217
218  public static Command createCommand(Configuration conf, BackupCommand type, CommandLine cmdline) {
219    Command cmd;
220    switch (type) {
221      case CREATE:
222        cmd = new CreateCommand(conf, cmdline);
223        break;
224      case DESCRIBE:
225        cmd = new DescribeCommand(conf, cmdline);
226        break;
227      case PROGRESS:
228        cmd = new ProgressCommand(conf, cmdline);
229        break;
230      case DELETE:
231        cmd = new DeleteCommand(conf, cmdline);
232        break;
233      case HISTORY:
234        cmd = new HistoryCommand(conf, cmdline);
235        break;
236      case SET:
237        cmd = new BackupSetCommand(conf, cmdline);
238        break;
239      case REPAIR:
240        cmd = new RepairCommand(conf, cmdline);
241        break;
242      case MERGE:
243        cmd = new MergeCommand(conf, cmdline);
244        break;
245      case HELP:
246      default:
247        cmd = new HelpCommand(conf, cmdline);
248        break;
249    }
250    return cmd;
251  }
252
253  static int numOfArgs(String[] args) {
254    if (args == null) {
255      return 0;
256    }
257
258    return args.length;
259  }
260
261  public static class CreateCommand extends Command {
262    CreateCommand(Configuration conf, CommandLine cmdline) {
263      super(conf);
264      this.cmdline = cmdline;
265    }
266
267    @Override
268    protected boolean requiresNoActiveSession() {
269      return true;
270    }
271
272    @Override
273    protected boolean requiresConsistentState() {
274      return true;
275    }
276
277    @Override
278    public void execute() throws IOException {
279      if (cmdline == null || cmdline.getArgs() == null) {
280        printUsage();
281        throw new IOException(INCORRECT_USAGE);
282      }
283      String[] args = cmdline.getArgs();
284      if (args.length != 3) {
285        printUsage();
286        throw new IOException(INCORRECT_USAGE);
287      }
288
289      if (
290        !BackupType.FULL.toString().equalsIgnoreCase(args[1])
291          && !BackupType.INCREMENTAL.toString().equalsIgnoreCase(args[1])
292      ) {
293        System.out.println("ERROR: invalid backup type: " + args[1]);
294        printUsage();
295        throw new IOException(INCORRECT_USAGE);
296      }
297      if (!verifyPath(args[2])) {
298        System.out.println("ERROR: invalid backup destination: " + args[2]);
299        printUsage();
300        throw new IOException(INCORRECT_USAGE);
301      }
302      String targetBackupDir = args[2];
303      // Check if backup destination is top level (root) folder - not allowed
304      if (isRootFolder(targetBackupDir)) {
305        throw new IOException(TOP_LEVEL_NOT_ALLOWED);
306      }
307      String tables;
308
309      // Check if we have both: backup set and list of tables
310      if (cmdline.hasOption(OPTION_TABLE) && cmdline.hasOption(OPTION_SET)) {
311        System.out
312          .println("ERROR: You can specify either backup set or list" + " of tables, but not both");
313        printUsage();
314        throw new IOException(INCORRECT_USAGE);
315      }
316      // Creates connection
317      super.execute();
318      // Check backup set
319      String setName = null;
320      if (cmdline.hasOption(OPTION_SET)) {
321        setName = cmdline.getOptionValue(OPTION_SET);
322        tables = getTablesForSet(setName);
323
324        if (tables == null) {
325          System.out
326            .println("ERROR: Backup set '" + setName + "' is either empty or does not exist");
327          printUsage();
328          throw new IOException(INCORRECT_USAGE);
329        }
330      } else {
331        tables = cmdline.getOptionValue(OPTION_TABLE);
332      }
333      int bandwidth = cmdline.hasOption(OPTION_BANDWIDTH)
334        ? Integer.parseInt(cmdline.getOptionValue(OPTION_BANDWIDTH))
335        : -1;
336      int workers = cmdline.hasOption(OPTION_WORKERS)
337        ? Integer.parseInt(cmdline.getOptionValue(OPTION_WORKERS))
338        : -1;
339
340      boolean ignoreChecksum = cmdline.hasOption(OPTION_IGNORECHECKSUM);
341
342      try (BackupAdminImpl admin = new BackupAdminImpl(conn)) {
343        BackupRequest.Builder builder = new BackupRequest.Builder();
344        BackupRequest request = builder.withBackupType(BackupType.valueOf(args[1].toUpperCase()))
345          .withTableList(
346            tables != null ? Lists.newArrayList(BackupUtils.parseTableNames(tables)) : null)
347          .withTargetRootDir(targetBackupDir).withTotalTasks(workers)
348          .withBandwidthPerTasks(bandwidth).withNoChecksumVerify(ignoreChecksum)
349          .withBackupSetName(setName).build();
350        String backupId = admin.backupTables(request);
351        System.out.println("Backup session " + backupId + " finished. Status: SUCCESS");
352      } catch (IOException e) {
353        System.out.println("Backup session finished. Status: FAILURE");
354        throw e;
355      }
356    }
357
358    private boolean isRootFolder(String targetBackupDir) {
359      Path p = new Path(targetBackupDir);
360      return p.isRoot();
361    }
362
363    private boolean verifyPath(String path) {
364      try {
365        Path p = new Path(path);
366        Configuration conf = getConf() != null ? getConf() : HBaseConfiguration.create();
367        URI uri = p.toUri();
368
369        if (uri.getScheme() == null) {
370          return false;
371        }
372
373        FileSystem.get(uri, conf);
374        return true;
375      } catch (Exception e) {
376        return false;
377      }
378    }
379
380    private String getTablesForSet(String name) throws IOException {
381      try (final BackupSystemTable table = new BackupSystemTable(conn)) {
382        List<TableName> tables = table.describeBackupSet(name);
383
384        if (tables == null) {
385          return null;
386        }
387
388        return StringUtils.join(tables, BackupRestoreConstants.TABLENAME_DELIMITER_IN_COMMAND);
389      }
390    }
391
392    @Override
393    protected void printUsage() {
394      System.out.println(CREATE_CMD_USAGE);
395      Options options = new Options();
396      options.addOption(OPTION_WORKERS, true, OPTION_WORKERS_DESC);
397      options.addOption(OPTION_BANDWIDTH, true, OPTION_BANDWIDTH_DESC);
398      options.addOption(OPTION_SET, true, OPTION_SET_BACKUP_DESC);
399      options.addOption(OPTION_TABLE, true, OPTION_TABLE_LIST_DESC);
400      options.addOption(OPTION_YARN_QUEUE_NAME, true, OPTION_YARN_QUEUE_NAME_DESC);
401      options.addOption(OPTION_DEBUG, false, OPTION_DEBUG_DESC);
402      options.addOption(OPTION_IGNORECHECKSUM, false, OPTION_IGNORECHECKSUM_DESC);
403
404      HelpFormatter helpFormatter = new HelpFormatter();
405      helpFormatter.setLeftPadding(2);
406      helpFormatter.setDescPadding(8);
407      helpFormatter.setWidth(100);
408      helpFormatter.setSyntaxPrefix("Options:");
409      helpFormatter.printHelp(" ", null, options, USAGE_FOOTER);
410    }
411  }
412
413  public static class HelpCommand extends Command {
414    HelpCommand(Configuration conf, CommandLine cmdline) {
415      super(conf);
416      this.cmdline = cmdline;
417    }
418
419    @Override
420    public void execute() throws IOException {
421      if (cmdline == null) {
422        printUsage();
423        throw new IOException(INCORRECT_USAGE);
424      }
425
426      String[] args = cmdline.getArgs();
427      if (args == null || args.length == 0) {
428        printUsage();
429        throw new IOException(INCORRECT_USAGE);
430      }
431
432      if (args.length != 2) {
433        System.out.println("ERROR: Only supports help message of a single command type");
434        printUsage();
435        throw new IOException(INCORRECT_USAGE);
436      }
437
438      String type = args[1];
439
440      if (BackupCommand.CREATE.name().equalsIgnoreCase(type)) {
441        System.out.println(CREATE_CMD_USAGE);
442      } else if (BackupCommand.DESCRIBE.name().equalsIgnoreCase(type)) {
443        System.out.println(DESCRIBE_CMD_USAGE);
444      } else if (BackupCommand.HISTORY.name().equalsIgnoreCase(type)) {
445        System.out.println(HISTORY_CMD_USAGE);
446      } else if (BackupCommand.PROGRESS.name().equalsIgnoreCase(type)) {
447        System.out.println(PROGRESS_CMD_USAGE);
448      } else if (BackupCommand.DELETE.name().equalsIgnoreCase(type)) {
449        System.out.println(DELETE_CMD_USAGE);
450      } else if (BackupCommand.SET.name().equalsIgnoreCase(type)) {
451        System.out.println(SET_CMD_USAGE);
452      } else {
453        System.out.println("Unknown command : " + type);
454        printUsage();
455      }
456    }
457
458    @Override
459    protected void printUsage() {
460      System.out.println(USAGE);
461    }
462  }
463
464  public static class DescribeCommand extends Command {
465    DescribeCommand(Configuration conf, CommandLine cmdline) {
466      super(conf);
467      this.cmdline = cmdline;
468    }
469
470    @Override
471    public void execute() throws IOException {
472      if (cmdline == null || cmdline.getArgs() == null) {
473        printUsage();
474        throw new IOException(INCORRECT_USAGE);
475      }
476      String[] args = cmdline.getArgs();
477      if (args.length != 2) {
478        printUsage();
479        throw new IOException(INCORRECT_USAGE);
480      }
481
482      super.execute();
483
484      String backupId = args[1];
485      try (final BackupSystemTable sysTable = new BackupSystemTable(conn)) {
486        BackupInfo info = sysTable.readBackupInfo(backupId);
487        if (info == null) {
488          System.out.println("ERROR: " + backupId + " does not exist");
489          printUsage();
490          throw new IOException(INCORRECT_USAGE);
491        }
492        System.out.println(info.getShortDescription());
493      }
494    }
495
496    @Override
497    protected void printUsage() {
498      System.out.println(DESCRIBE_CMD_USAGE);
499    }
500  }
501
502  public static class ProgressCommand extends Command {
503    ProgressCommand(Configuration conf, CommandLine cmdline) {
504      super(conf);
505      this.cmdline = cmdline;
506    }
507
508    @Override
509    public void execute() throws IOException {
510
511      if (cmdline == null || cmdline.getArgs() == null || cmdline.getArgs().length == 1) {
512        System.out.println(
513          "No backup id was specified, " + "will retrieve the most recent (ongoing) session");
514      }
515      String[] args = cmdline == null ? null : cmdline.getArgs();
516      if (args != null && args.length > 2) {
517        System.err.println("ERROR: wrong number of arguments: " + args.length);
518        printUsage();
519        throw new IOException(INCORRECT_USAGE);
520      }
521
522      super.execute();
523
524      String backupId = (args == null || args.length <= 1) ? null : args[1];
525      try (final BackupSystemTable sysTable = new BackupSystemTable(conn)) {
526        BackupInfo info = null;
527
528        if (backupId != null) {
529          info = sysTable.readBackupInfo(backupId);
530        } else {
531          List<BackupInfo> infos = sysTable.getBackupInfos(BackupState.RUNNING);
532          if (infos != null && infos.size() > 0) {
533            info = infos.get(0);
534            backupId = info.getBackupId();
535            System.out.println("Found ongoing session with backupId=" + backupId);
536          }
537        }
538        int progress = info == null ? -1 : info.getProgress();
539        if (progress < 0) {
540          if (backupId != null) {
541            System.out.println(NO_INFO_FOUND + backupId);
542          } else {
543            System.err.println(NO_ACTIVE_SESSION_FOUND);
544          }
545        } else {
546          System.out.println(backupId + " progress=" + progress + "%");
547        }
548      }
549    }
550
551    @Override
552    protected void printUsage() {
553      System.out.println(PROGRESS_CMD_USAGE);
554    }
555  }
556
557  public static class DeleteCommand extends Command {
558    DeleteCommand(Configuration conf, CommandLine cmdline) {
559      super(conf);
560      this.cmdline = cmdline;
561    }
562
563    @Override
564    protected boolean requiresNoActiveSession() {
565      return true;
566    }
567
568    @Override
569    public void execute() throws IOException {
570
571      if (cmdline == null || cmdline.getArgs() == null || cmdline.getArgs().length < 1) {
572        printUsage();
573        throw new IOException(INCORRECT_USAGE);
574      }
575
576      if (!cmdline.hasOption(OPTION_KEEP) && !cmdline.hasOption(OPTION_LIST)) {
577        printUsage();
578        throw new IOException(INCORRECT_USAGE);
579      }
580      super.execute();
581      if (cmdline.hasOption(OPTION_KEEP)) {
582        executeDeleteOlderThan(cmdline);
583      } else if (cmdline.hasOption(OPTION_LIST)) {
584        executeDeleteListOfBackups(cmdline);
585      }
586    }
587
588    private void executeDeleteOlderThan(CommandLine cmdline) throws IOException {
589      String value = cmdline.getOptionValue(OPTION_KEEP);
590      int days = 0;
591      try {
592        days = Integer.parseInt(value);
593      } catch (NumberFormatException e) {
594        throw new IOException(value + " is not an integer number");
595      }
596      final long fdays = days;
597      BackupInfo.Filter dateFilter = new BackupInfo.Filter() {
598        @Override
599        public boolean apply(BackupInfo info) {
600          long currentTime = EnvironmentEdgeManager.currentTime();
601          long maxTsToDelete = currentTime - fdays * 24 * 3600 * 1000;
602          return info.getCompleteTs() <= maxTsToDelete;
603        }
604      };
605      List<BackupInfo> history = null;
606      try (final BackupSystemTable sysTable = new BackupSystemTable(conn);
607        BackupAdminImpl admin = new BackupAdminImpl(conn)) {
608        history = sysTable.getBackupHistory(-1, dateFilter);
609        String[] backupIds = convertToBackupIds(history);
610        int deleted = admin.deleteBackups(backupIds);
611        System.out.println("Deleted " + deleted + " backups. Total older than " + days + " days: "
612          + backupIds.length);
613      } catch (IOException e) {
614        System.err.println("Delete command FAILED. Please run backup repair tool to restore backup "
615          + "system integrity");
616        throw e;
617      }
618    }
619
620    private String[] convertToBackupIds(List<BackupInfo> history) {
621      String[] ids = new String[history.size()];
622      for (int i = 0; i < ids.length; i++) {
623        ids[i] = history.get(i).getBackupId();
624      }
625      return ids;
626    }
627
628    private void executeDeleteListOfBackups(CommandLine cmdline) throws IOException {
629      String value = cmdline.getOptionValue(OPTION_LIST);
630      String[] backupIds = value.split(",");
631
632      try (BackupAdminImpl admin = new BackupAdminImpl(conn)) {
633        int deleted = admin.deleteBackups(backupIds);
634        System.out.println("Deleted " + deleted + " backups. Total requested: " + backupIds.length);
635      } catch (IOException e) {
636        System.err.println("Delete command FAILED. Please run backup repair tool to restore backup "
637          + "system integrity");
638        throw e;
639      }
640
641    }
642
643    @Override
644    protected void printUsage() {
645      System.out.println(DELETE_CMD_USAGE);
646      Options options = new Options();
647      options.addOption(OPTION_KEEP, true, OPTION_KEEP_DESC);
648      options.addOption(OPTION_LIST, true, OPTION_BACKUP_LIST_DESC);
649
650      HelpFormatter helpFormatter = new HelpFormatter();
651      helpFormatter.setLeftPadding(2);
652      helpFormatter.setDescPadding(8);
653      helpFormatter.setWidth(100);
654      helpFormatter.setSyntaxPrefix("Options:");
655      helpFormatter.printHelp(" ", null, options, USAGE_FOOTER);
656
657    }
658  }
659
660  public static class RepairCommand extends Command {
661    RepairCommand(Configuration conf, CommandLine cmdline) {
662      super(conf);
663      this.cmdline = cmdline;
664    }
665
666    @Override
667    public void execute() throws IOException {
668      super.execute();
669
670      String[] args = cmdline == null ? null : cmdline.getArgs();
671      if (args != null && args.length > 1) {
672        System.err.println("ERROR: wrong number of arguments: " + args.length);
673        printUsage();
674        throw new IOException(INCORRECT_USAGE);
675      }
676
677      Configuration conf = getConf() != null ? getConf() : HBaseConfiguration.create();
678      try (final Connection conn = ConnectionFactory.createConnection(conf);
679        final BackupSystemTable sysTable = new BackupSystemTable(conn)) {
680        // Failed backup
681        BackupInfo backupInfo;
682        List<BackupInfo> list = sysTable.getBackupInfos(BackupState.RUNNING);
683        if (list.size() == 0) {
684          // No failed sessions found
685          System.out.println("REPAIR status: no failed sessions found."
686            + " Checking failed delete backup operation ...");
687          repairFailedBackupDeletionIfAny(conn, sysTable);
688          repairFailedBackupMergeIfAny(conn, sysTable);
689          return;
690        }
691        backupInfo = list.get(0);
692        // If this is a cancel exception, then we've already cleaned.
693        // set the failure timestamp of the overall backup
694        backupInfo.setCompleteTs(EnvironmentEdgeManager.currentTime());
695        // set failure message
696        backupInfo.setFailedMsg("REPAIR status: repaired after failure:\n" + backupInfo);
697        // set overall backup status: failed
698        backupInfo.setState(BackupState.FAILED);
699        // compose the backup failed data
700        String backupFailedData = "BackupId=" + backupInfo.getBackupId() + ",startts="
701          + backupInfo.getStartTs() + ",failedts=" + backupInfo.getCompleteTs() + ",failedphase="
702          + backupInfo.getPhase() + ",failedmessage=" + backupInfo.getFailedMsg();
703        System.out.println(backupFailedData);
704        TableBackupClient.cleanupAndRestoreBackupSystem(conn, backupInfo, conf);
705        // If backup session is updated to FAILED state - means we
706        // processed recovery already.
707        sysTable.updateBackupInfo(backupInfo);
708        sysTable.finishBackupExclusiveOperation();
709        System.out.println("REPAIR status: finished repair failed session:\n " + backupInfo);
710      }
711    }
712
713    private void repairFailedBackupDeletionIfAny(Connection conn, BackupSystemTable sysTable)
714      throws IOException {
715      String[] backupIds = sysTable.getListOfBackupIdsFromDeleteOperation();
716      if (backupIds == null || backupIds.length == 0) {
717        System.out.println("No failed backup DELETE operation found");
718        // Delete backup table snapshot if exists
719        BackupSystemTable.deleteSnapshot(conn);
720        return;
721      }
722      System.out.println("Found failed DELETE operation for: " + StringUtils.join(backupIds));
723      System.out.println("Running DELETE again ...");
724      // Restore table from snapshot
725      BackupSystemTable.restoreFromSnapshot(conn);
726      // Finish previous failed session
727      sysTable.finishBackupExclusiveOperation();
728      try (BackupAdmin admin = new BackupAdminImpl(conn)) {
729        admin.deleteBackups(backupIds);
730      }
731      System.out.println("DELETE operation finished OK: " + StringUtils.join(backupIds));
732    }
733
734    public static void repairFailedBackupMergeIfAny(Connection conn, BackupSystemTable sysTable)
735      throws IOException {
736
737      String[] backupIds = sysTable.getListOfBackupIdsFromMergeOperation();
738      if (backupIds == null || backupIds.length == 0) {
739        System.out.println("No failed backup MERGE operation found");
740        // Delete backup table snapshot if exists
741        BackupSystemTable.deleteSnapshot(conn);
742        return;
743      }
744      System.out.println("Found failed MERGE operation for: " + StringUtils.join(backupIds));
745      // Check if backup .tmp exists
746      BackupInfo bInfo = sysTable.readBackupInfo(backupIds[0]);
747      String backupRoot = bInfo.getBackupRootDir();
748      FileSystem fs = FileSystem.get(new Path(backupRoot).toUri(), new Configuration());
749      String backupId = BackupUtils.findMostRecentBackupId(backupIds);
750      Path tmpPath = HBackupFileSystem.getBackupTmpDirPathForBackupId(backupRoot, backupId);
751      if (fs.exists(tmpPath)) {
752        // Move data back
753        Path destPath = HBackupFileSystem.getBackupPath(backupRoot, backupId);
754        if (!fs.delete(destPath, true)) {
755          System.out.println("Failed to delete " + destPath);
756        }
757        boolean res = fs.rename(tmpPath, destPath);
758        if (!res) {
759          throw new IOException(
760            "MERGE repair: failed  to rename from " + tmpPath + " to " + destPath);
761        }
762        System.out
763          .println("MERGE repair: renamed from " + tmpPath + " to " + destPath + " res=" + res);
764      } else {
765        checkRemoveBackupImages(fs, backupRoot, backupIds);
766      }
767      // Restore table from snapshot
768      BackupSystemTable.restoreFromSnapshot(conn);
769      // Unlock backup system
770      sysTable.finishBackupExclusiveOperation();
771      // Finish previous failed session
772      sysTable.finishMergeOperation();
773
774      System.out.println("MERGE repair operation finished OK: " + StringUtils.join(backupIds));
775    }
776
777    private static void checkRemoveBackupImages(FileSystem fs, String backupRoot,
778      String[] backupIds) throws IOException {
779      String mergedBackupId = BackupUtils.findMostRecentBackupId(backupIds);
780      for (String backupId : backupIds) {
781        if (backupId.equals(mergedBackupId)) {
782          continue;
783        }
784        Path path = HBackupFileSystem.getBackupPath(backupRoot, backupId);
785        if (fs.exists(path)) {
786          if (!fs.delete(path, true)) {
787            System.out.println("MERGE repair removing: " + path + " - FAILED");
788          } else {
789            System.out.println("MERGE repair removing: " + path + " - OK");
790          }
791        }
792      }
793    }
794
795    @Override
796    protected void printUsage() {
797      System.out.println(REPAIR_CMD_USAGE);
798    }
799  }
800
801  public static class MergeCommand extends Command {
802    MergeCommand(Configuration conf, CommandLine cmdline) {
803      super(conf);
804      this.cmdline = cmdline;
805    }
806
807    @Override
808    protected boolean requiresNoActiveSession() {
809      return true;
810    }
811
812    @Override
813    protected boolean requiresConsistentState() {
814      return true;
815    }
816
817    @Override
818    public void execute() throws IOException {
819      super.execute();
820
821      String[] args = cmdline == null ? null : cmdline.getArgs();
822      if (args == null || (args.length != 2)) {
823        System.err
824          .println("ERROR: wrong number of arguments: " + (args == null ? null : args.length));
825        printUsage();
826        throw new IOException(INCORRECT_USAGE);
827      }
828
829      String[] backupIds = args[1].split(",");
830      if (backupIds.length < 2) {
831        String msg = "ERROR: can not merge a single backup image. "
832          + "Number of images must be greater than 1.";
833        System.err.println(msg);
834        throw new IOException(msg);
835
836      }
837      Configuration conf = getConf() != null ? getConf() : HBaseConfiguration.create();
838      try (final Connection conn = ConnectionFactory.createConnection(conf);
839        final BackupAdminImpl admin = new BackupAdminImpl(conn)) {
840        admin.mergeBackups(backupIds);
841      }
842    }
843
844    @Override
845    protected void printUsage() {
846      System.out.println(MERGE_CMD_USAGE);
847    }
848  }
849
850  public static class HistoryCommand extends Command {
851    private final static int DEFAULT_HISTORY_LENGTH = 10;
852
853    HistoryCommand(Configuration conf, CommandLine cmdline) {
854      super(conf);
855      this.cmdline = cmdline;
856    }
857
858    @Override
859    public void execute() throws IOException {
860      int n = parseHistoryLength();
861      final TableName tableName = getTableName();
862      final String setName = getTableSetName();
863      BackupInfo.Filter tableNameFilter = new BackupInfo.Filter() {
864        @Override
865        public boolean apply(BackupInfo info) {
866          if (tableName == null) {
867            return true;
868          }
869
870          List<TableName> names = info.getTableNames();
871          return names.contains(tableName);
872        }
873      };
874      BackupInfo.Filter tableSetFilter = new BackupInfo.Filter() {
875        @Override
876        public boolean apply(BackupInfo info) {
877          if (setName == null) {
878            return true;
879          }
880
881          String backupId = info.getBackupId();
882          return backupId.startsWith(setName);
883        }
884      };
885      Path backupRootPath = getBackupRootPath();
886      List<BackupInfo> history;
887      if (backupRootPath == null) {
888        // Load from backup system table
889        super.execute();
890        try (final BackupSystemTable sysTable = new BackupSystemTable(conn)) {
891          history = sysTable.getBackupHistory(n, tableNameFilter, tableSetFilter);
892        }
893      } else {
894        // load from backup FS
895        history =
896          BackupUtils.getHistory(getConf(), n, backupRootPath, tableNameFilter, tableSetFilter);
897      }
898      for (BackupInfo info : history) {
899        System.out.println(info.getShortDescription());
900      }
901    }
902
903    private Path getBackupRootPath() throws IOException {
904      String value = null;
905      try {
906        value = cmdline.getOptionValue(OPTION_PATH);
907
908        if (value == null) {
909          return null;
910        }
911
912        return new Path(value);
913      } catch (IllegalArgumentException e) {
914        System.out.println("ERROR: Illegal argument for backup root path: " + value);
915        printUsage();
916        throw new IOException(INCORRECT_USAGE);
917      }
918    }
919
920    private TableName getTableName() throws IOException {
921      String value = cmdline.getOptionValue(OPTION_TABLE);
922
923      if (value == null) {
924        return null;
925      }
926
927      try {
928        return TableName.valueOf(value);
929      } catch (IllegalArgumentException e) {
930        System.out.println("Illegal argument for table name: " + value);
931        printUsage();
932        throw new IOException(INCORRECT_USAGE);
933      }
934    }
935
936    private String getTableSetName() {
937      return cmdline.getOptionValue(OPTION_SET);
938    }
939
940    private int parseHistoryLength() throws IOException {
941      String value = cmdline.getOptionValue(OPTION_RECORD_NUMBER);
942      try {
943        if (value == null) {
944          return DEFAULT_HISTORY_LENGTH;
945        }
946
947        return Integer.parseInt(value);
948      } catch (NumberFormatException e) {
949        System.out.println("Illegal argument for history length: " + value);
950        printUsage();
951        throw new IOException(INCORRECT_USAGE);
952      }
953    }
954
955    @Override
956    protected void printUsage() {
957      System.out.println(HISTORY_CMD_USAGE);
958      Options options = new Options();
959      options.addOption(OPTION_RECORD_NUMBER, true, OPTION_RECORD_NUMBER_DESC);
960      options.addOption(OPTION_PATH, true, OPTION_PATH_DESC);
961      options.addOption(OPTION_TABLE, true, OPTION_TABLE_DESC);
962      options.addOption(OPTION_SET, true, OPTION_SET_DESC);
963
964      HelpFormatter helpFormatter = new HelpFormatter();
965      helpFormatter.setLeftPadding(2);
966      helpFormatter.setDescPadding(8);
967      helpFormatter.setWidth(100);
968      helpFormatter.setSyntaxPrefix("Options:");
969      helpFormatter.printHelp(" ", null, options, USAGE_FOOTER);
970    }
971  }
972
973  public static class BackupSetCommand extends Command {
974    private final static String SET_ADD_CMD = "add";
975    private final static String SET_REMOVE_CMD = "remove";
976    private final static String SET_DELETE_CMD = "delete";
977    private final static String SET_DESCRIBE_CMD = "describe";
978    private final static String SET_LIST_CMD = "list";
979
980    BackupSetCommand(Configuration conf, CommandLine cmdline) {
981      super(conf);
982      this.cmdline = cmdline;
983    }
984
985    @Override
986    public void execute() throws IOException {
987      // Command-line must have at least one element
988      if (cmdline == null || cmdline.getArgs() == null || cmdline.getArgs().length < 2) {
989        printUsage();
990        throw new IOException(INCORRECT_USAGE);
991      }
992
993      String[] args = cmdline.getArgs();
994      String cmdStr = args[1];
995      BackupCommand cmd = getCommand(cmdStr);
996
997      switch (cmd) {
998        case SET_ADD:
999          processSetAdd(args);
1000          break;
1001        case SET_REMOVE:
1002          processSetRemove(args);
1003          break;
1004        case SET_DELETE:
1005          processSetDelete(args);
1006          break;
1007        case SET_DESCRIBE:
1008          processSetDescribe(args);
1009          break;
1010        case SET_LIST:
1011          processSetList();
1012          break;
1013        default:
1014          break;
1015      }
1016    }
1017
1018    private void processSetList() throws IOException {
1019      super.execute();
1020
1021      // List all backup set names
1022      // does not expect any args
1023      try (BackupAdminImpl admin = new BackupAdminImpl(conn)) {
1024        List<BackupSet> list = admin.listBackupSets();
1025        for (BackupSet bs : list) {
1026          System.out.println(bs);
1027        }
1028      }
1029    }
1030
1031    private void processSetDescribe(String[] args) throws IOException {
1032      if (args == null || args.length != 3) {
1033        printUsage();
1034        throw new IOException(INCORRECT_USAGE);
1035      }
1036      super.execute();
1037
1038      String setName = args[2];
1039      try (final BackupSystemTable sysTable = new BackupSystemTable(conn)) {
1040        List<TableName> tables = sysTable.describeBackupSet(setName);
1041        BackupSet set = tables == null ? null : new BackupSet(setName, tables);
1042        if (set == null) {
1043          System.out.println("Set '" + setName + "' does not exist.");
1044        } else {
1045          System.out.println(set);
1046        }
1047      }
1048    }
1049
1050    private void processSetDelete(String[] args) throws IOException {
1051      if (args == null || args.length != 3) {
1052        printUsage();
1053        throw new IOException(INCORRECT_USAGE);
1054      }
1055      super.execute();
1056
1057      String setName = args[2];
1058      try (final BackupAdminImpl admin = new BackupAdminImpl(conn)) {
1059        boolean result = admin.deleteBackupSet(setName);
1060        if (result) {
1061          System.out.println("Delete set " + setName + " OK.");
1062        } else {
1063          System.out.println("Set " + setName + " does not exist");
1064        }
1065      }
1066    }
1067
1068    private void processSetRemove(String[] args) throws IOException {
1069      if (args == null || args.length != 4) {
1070        printUsage();
1071        throw new IOException(INCORRECT_USAGE);
1072      }
1073      super.execute();
1074
1075      String setName = args[2];
1076      String[] tables = args[3].split(",");
1077      TableName[] tableNames = toTableNames(tables);
1078      try (final BackupAdminImpl admin = new BackupAdminImpl(conn)) {
1079        admin.removeFromBackupSet(setName, tableNames);
1080      }
1081    }
1082
1083    private TableName[] toTableNames(String[] tables) {
1084      TableName[] arr = new TableName[tables.length];
1085      for (int i = 0; i < tables.length; i++) {
1086        arr[i] = TableName.valueOf(tables[i]);
1087      }
1088      return arr;
1089    }
1090
1091    private void processSetAdd(String[] args) throws IOException {
1092      if (args == null || args.length != 4) {
1093        printUsage();
1094        throw new IOException(INCORRECT_USAGE);
1095      }
1096      super.execute();
1097      String setName = args[2];
1098      TableName[] tableNames =
1099        Splitter.on(',').splitToStream(args[3]).map(TableName::valueOf).toArray(TableName[]::new);
1100      try (final BackupAdminImpl admin = new BackupAdminImpl(conn)) {
1101        admin.addToBackupSet(setName, tableNames);
1102      }
1103    }
1104
1105    private BackupCommand getCommand(String cmdStr) throws IOException {
1106      switch (cmdStr) {
1107        case SET_ADD_CMD:
1108          return BackupCommand.SET_ADD;
1109        case SET_REMOVE_CMD:
1110          return BackupCommand.SET_REMOVE;
1111        case SET_DELETE_CMD:
1112          return BackupCommand.SET_DELETE;
1113        case SET_DESCRIBE_CMD:
1114          return BackupCommand.SET_DESCRIBE;
1115        case SET_LIST_CMD:
1116          return BackupCommand.SET_LIST;
1117        default:
1118          System.out.println("ERROR: Unknown command for 'set' :" + cmdStr);
1119          printUsage();
1120          throw new IOException(INCORRECT_USAGE);
1121      }
1122    }
1123
1124    @Override
1125    protected void printUsage() {
1126      System.out.println(SET_CMD_USAGE);
1127    }
1128  }
1129}