Skip to content

Commit

Permalink
Backup 2.0 Restore
Browse files Browse the repository at this point in the history
  • Loading branch information
arunagrawal84 committed Jan 11, 2019
1 parent bb14098 commit 43c7f1c
Show file tree
Hide file tree
Showing 27 changed files with 712 additions and 313 deletions.
10 changes: 10 additions & 0 deletions CHANGELOG.md
Original file line number Diff line number Diff line change
@@ -1,4 +1,14 @@
# Changelog

## 2018/01/11 3.11.38
(#761) Add new file format (SST_V2) and methods to get/parse remote locations.
(#761) Upload files from SnapshotMetaService in backup version 2.0, if enabled.
(#761) Process older SNAPSHOT_V2 at the restart of Priam.
(#767) Backup Verification for Backup 2.0.
(#767) Restore for Backup 2.0
(#767) Some API changes for Snapshot Verification
(#767) Remove deprecated code like flush hour or snapshot hour.

## 2018/10/29 3.11.37
* Bug Fix: SnapshotMetaService can leave snapshots if there is any error.
* Bug Fix: SnapshotMetaService should continue building snapshot even if an unexpected file is found in snapshot.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -106,7 +106,6 @@ protected List<AbstractBackupPath> upload(
true);

bps.add(bp);
addToRemotePath(bp.getRemotePath());
}
}

Expand Down Expand Up @@ -183,7 +182,4 @@ private boolean isValidBackupDir(File keyspaceDir, File backupDir) {

return true;
}

/** Adds Remote path to the list of Remote Paths */
protected abstract void addToRemotePath(String remotePath);
}
Original file line number Diff line number Diff line change
Expand Up @@ -115,8 +115,8 @@ public Future<Path> asyncDownloadFile(
public void downloadFile(final Path remotePath, final Path localPath, final int retry)
throws BackupRestoreException {
// TODO: Should we download the file if localPath already exists?
if (remotePath == null) return;

if (remotePath == null || localPath == null) return;
localPath.toFile().getParentFile().mkdirs();
logger.info("Downloading file: {} to location: {}", remotePath, localPath);
try {
new BoundedExponentialRetryCallable<Void>(500, 10000, retry) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,6 @@
import java.nio.file.Path;
import java.nio.file.Paths;
import java.util.*;
import java.util.stream.Collectors;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand All @@ -46,22 +45,21 @@ public class BackupVerification {
}

public Optional<BackupMetadata> getLatestBackupMetaData(List<BackupMetadata> metadata) {
metadata =
metadata.stream()
.filter(backupMetadata -> backupMetadata.getStatus() == Status.FINISHED)
.collect(Collectors.toList());
metadata.sort((o1, o2) -> o2.getStart().compareTo(o1.getStart()));
return metadata.stream().findFirst();
return metadata.stream()
.filter(backupMetadata -> backupMetadata != null)
.filter(backupMetadata -> backupMetadata.getStatus() == Status.FINISHED)
.sorted(Comparator.comparing(BackupMetadata::getStart).reversed())
.findFirst();
}

public Optional<BackupVerificationResult> verifyBackup(List<BackupMetadata> metadata) {
if (metadata == null || metadata.isEmpty()) return null;
if (metadata == null || metadata.isEmpty()) return Optional.empty();

Optional<BackupMetadata> latestBackupMetaData = getLatestBackupMetaData(metadata);

if (!latestBackupMetaData.isPresent()) {
logger.error("No backup found which finished during the time provided.");
return null;
return Optional.empty();
}

Path metadataLocation = Paths.get(latestBackupMetaData.get().getSnapshotLocation());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,8 +20,6 @@
import com.netflix.priam.scheduler.SimpleTimer;
import com.netflix.priam.scheduler.TaskTimer;
import java.io.File;
import java.util.ArrayList;
import java.util.List;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand All @@ -31,7 +29,6 @@ public class CommitLogBackupTask extends AbstractBackup {
public static final String JOBNAME = "CommitLogBackup";

private static final Logger logger = LoggerFactory.getLogger(CommitLogBackupTask.class);
private final List<String> clRemotePaths = new ArrayList<>();
private final CommitLogBackup clBackup;

@Inject
Expand Down Expand Up @@ -70,9 +67,4 @@ protected void processColumnFamily(String keyspace, String columnFamily, File ba
throws Exception {
// Do nothing.
}

@Override
protected void addToRemotePath(String remotePath) {
clRemotePaths.add(remotePath);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -20,12 +20,12 @@
import com.google.inject.Provider;
import com.google.inject.Singleton;
import com.netflix.priam.backup.AbstractBackupPath.BackupFileType;
import com.netflix.priam.config.IBackupRestoreConfig;
import com.netflix.priam.config.IConfiguration;
import com.netflix.priam.scheduler.SimpleTimer;
import com.netflix.priam.scheduler.TaskTimer;
import com.netflix.priam.utils.DateUtil;
import java.io.File;
import java.util.ArrayList;
import java.util.List;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
Expand All @@ -37,20 +37,22 @@
public class IncrementalBackup extends AbstractBackup {
private static final Logger logger = LoggerFactory.getLogger(IncrementalBackup.class);
public static final String JOBNAME = "IncrementalBackup";
private final List<String> incrementalRemotePaths = new ArrayList<>();
private final IncrementalMetaData metaData;
private final BackupRestoreUtil backupRestoreUtil;
private final IBackupRestoreConfig backupRestoreConfig;

@Inject
public IncrementalBackup(
IConfiguration config,
IBackupRestoreConfig backupRestoreConfig,
Provider<AbstractBackupPath> pathFactory,
IFileSystemContext backupFileSystemCtx,
IncrementalMetaData metaData) {
super(config, backupFileSystemCtx, pathFactory);
// a means to upload audit trail (via meta_cf_yyyymmddhhmm.json) of files successfully
// uploaded)
this.metaData = metaData;
this.backupRestoreConfig = backupRestoreConfig;
backupRestoreUtil =
new BackupRestoreUtil(
config.getIncrementalIncludeCFList(), config.getIncrementalExcludeCFList());
Expand All @@ -59,7 +61,6 @@ public IncrementalBackup(
@Override
public void execute() throws Exception {
// Clearing remotePath List
incrementalRemotePaths.clear();
initiateBackup(INCREMENTAL_BACKUP_FOLDER, backupRestoreUtil);
}

Expand All @@ -76,8 +77,11 @@ public String getName() {
@Override
protected void processColumnFamily(String keyspace, String columnFamily, File backupDir)
throws Exception {
BackupFileType fileType = BackupFileType.SST;
if (backupRestoreConfig.enableV2Backups()) fileType = BackupFileType.SST_V2;

List<AbstractBackupPath> uploadedFiles =
upload(backupDir, BackupFileType.SST, config.enableAsyncIncremental(), true);
upload(backupDir, fileType, config.enableAsyncIncremental(), true);

if (!uploadedFiles.isEmpty()) {
// format of yyyymmddhhmm (e.g. 201505060901)
Expand All @@ -90,9 +94,4 @@ protected void processColumnFamily(String keyspace, String columnFamily, File ba
logger.info("Uploaded meta file for incremental backup: {}", metaFileName);
}
}

@Override
protected void addToRemotePath(String remotePath) {
incrementalRemotePaths.add(remotePath);
}
}
29 changes: 0 additions & 29 deletions priam/src/main/java/com/netflix/priam/backup/MetaData.java
Original file line number Diff line number Diff line change
Expand Up @@ -16,14 +16,12 @@
*/
package com.netflix.priam.backup;

import com.google.common.collect.Lists;
import com.google.inject.Inject;
import com.google.inject.Provider;
import com.netflix.priam.backup.AbstractBackupPath.BackupFileType;
import com.netflix.priam.config.IConfiguration;
import com.netflix.priam.utils.DateUtil;
import java.io.File;
import java.io.FileReader;
import java.io.FileWriter;
import java.io.IOException;
import java.nio.file.Paths;
Expand All @@ -32,7 +30,6 @@
import java.util.List;
import org.apache.commons.io.FileUtils;
import org.json.simple.JSONArray;
import org.json.simple.parser.JSONParser;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down Expand Up @@ -117,30 +114,4 @@ public File createTmpMetaFile() throws IOException {
private void addToRemotePath(String remotePath) {
metaRemotePaths.add(remotePath);
}

public List<AbstractBackupPath> toJson(File input) {
List<AbstractBackupPath> files = Lists.newArrayList();
try {
JSONArray jsonObj = (JSONArray) new JSONParser().parse(new FileReader(input));
for (Object aJsonObj : jsonObj) {
AbstractBackupPath p = pathFactory.get();
p.parseRemote((String) aJsonObj);
files.add(p);
}

} catch (Exception ex) {
throw new RuntimeException(
"Error transforming file "
+ input.getAbsolutePath()
+ " to JSON format. Msg:"
+ ex.getLocalizedMessage(),
ex);
}

logger.debug(
"Transformed file {} to JSON. Number of JSON elements: {}",
input.getAbsolutePath(),
files.size());
return files;
}
}
79 changes: 9 additions & 70 deletions priam/src/main/java/com/netflix/priam/backup/SnapshotBackup.java
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@
import com.google.inject.Provider;
import com.google.inject.Singleton;
import com.netflix.priam.backup.AbstractBackupPath.BackupFileType;
import com.netflix.priam.backupv2.ForgottenFilesManager;
import com.netflix.priam.config.IConfiguration;
import com.netflix.priam.defaultimpl.CassandraOperations;
import com.netflix.priam.identity.InstanceIdentity;
Expand All @@ -30,6 +31,7 @@
import com.netflix.priam.utils.DateUtil;
import com.netflix.priam.utils.ThreadSleeper;
import java.io.File;
import java.time.Instant;
import java.util.*;
import java.util.concurrent.locks.Lock;
import java.util.concurrent.locks.ReentrantLock;
Expand All @@ -42,13 +44,14 @@ public class SnapshotBackup extends AbstractBackup {
private static final Logger logger = LoggerFactory.getLogger(SnapshotBackup.class);
public static final String JOBNAME = "SnapshotBackup";
private final MetaData metaData;
private final List<String> snapshotRemotePaths = new ArrayList<>();
private final ThreadSleeper sleeper = new ThreadSleeper();
private static final long WAIT_TIME_MS = 60 * 1000 * 10;
private final InstanceIdentity instanceIdentity;
private final IBackupStatusMgr snapshotStatusMgr;
private final BackupRestoreUtil backupRestoreUtil;
private final ForgottenFilesManager forgottenFilesManager;
private String snapshotName = null;
private Instant snapshotInstant = DateUtil.getInstant();
private List<AbstractBackupPath> abstractBackupPaths = null;
private final CassandraOperations cassandraOperations;
private static final Lock lock = new ReentrantLock();
Expand All @@ -61,7 +64,8 @@ public SnapshotBackup(
IFileSystemContext backupFileSystemCtx,
IBackupStatusMgr snapshotStatusMgr,
InstanceIdentity instanceIdentity,
CassandraOperations cassandraOperations) {
CassandraOperations cassandraOperations,
ForgottenFilesManager forgottenFilesManager) {
super(config, backupFileSystemCtx, pathFactory);
this.metaData = metaData;
this.snapshotStatusMgr = snapshotStatusMgr;
Expand All @@ -70,6 +74,7 @@ public SnapshotBackup(
backupRestoreUtil =
new BackupRestoreUtil(
config.getSnapshotIncludeCFList(), config.getSnapshotExcludeCFList());
this.forgottenFilesManager = forgottenFilesManager;
}

@Override
Expand Down Expand Up @@ -100,6 +105,7 @@ public void execute() throws Exception {
private void executeSnapshot() throws Exception {
Date startTime = Calendar.getInstance(TimeZone.getTimeZone("GMT")).getTime();
snapshotName = DateUtil.formatyyyyMMddHHmm(startTime);
snapshotInstant = DateUtil.getInstant();
String token = instanceIdentity.getInstance().getToken();

// Save start snapshot status
Expand All @@ -108,8 +114,6 @@ private void executeSnapshot() throws Exception {

try {
logger.info("Starting snapshot {}", snapshotName);
// Clearing remotePath List
snapshotRemotePaths.clear();
cassandraOperations.takeSnapshot(snapshotName);

// Collect all snapshot dir's under keyspace dir's
Expand Down Expand Up @@ -177,74 +181,9 @@ protected void processColumnFamily(String keyspace, String columnFamily, File ba
return;
}

forgottenFilesManager.findAndMoveForgottenFiles(snapshotInstant, snapshotDir);
// Add files to this dir
abstractBackupPaths.addAll(
upload(snapshotDir, BackupFileType.SNAP, config.enableAsyncSnapshot(), true));
}

// private void findForgottenFiles(File snapshotDir) {
// try {
// Collection<File> snapshotFiles = FileUtils.listFiles(snapshotDir,
// FileFilterUtils.fileFileFilter(), null);
// File columnfamilyDir = snapshotDir.getParentFile().getParentFile();
//
// //Find all the files in columnfamily folder which is :
// // 1. Not a temp file.
// // 2. Is a file. (we don't care about directories)
// // 3. Is older than snapshot time, as new files keep getting created after taking
// a snapshot.
// IOFileFilter tmpFileFilter1 = FileFilterUtils.suffixFileFilter(TMP_EXT);
// IOFileFilter tmpFileFilter2 = FileFilterUtils.asFileFilter(pathname ->
// tmpFilePattern.matcher(pathname.getName()).matches());
// IOFileFilter tmpFileFilter = FileFilterUtils.or(tmpFileFilter1, tmpFileFilter2);
// // Here we are allowing files which were more than
// @link{IConfiguration#getForgottenFileGracePeriodDays}. We do this to allow cassandra to
// // clean up any files which were generated as part of repair/compaction and
// cleanup thread has not already deleted.
// // Refer to https://issues.apache.org/jira/browse/CASSANDRA-6756 and
// https://issues.apache.org/jira/browse/CASSANDRA-7066
// // for more information.
// IOFileFilter ageFilter =
// FileFilterUtils.ageFileFilter(snapshotInstant.minus(config.getForgottenFileGracePeriodDays(),
// ChronoUnit.DAYS).toEpochMilli());
// IOFileFilter fileFilter =
// FileFilterUtils.and(FileFilterUtils.notFileFilter(tmpFileFilter),
// FileFilterUtils.fileFileFilter(), ageFilter);
//
// Collection<File> columnfamilyFiles = FileUtils.listFiles(columnfamilyDir,
// fileFilter, null);
//
// //Remove the SSTable(s) which are part of snapshot from the CF file list.
// //This cannot be a simple removeAll as snapshot files have "different" file folder
// prefix.
// for (File file : snapshotFiles) {
// //Get its parent directory file based on this file.
// File originalFile = new File(columnfamilyDir, file.getName());
// columnfamilyFiles.remove(originalFile);
// }
//
// //If there are no "extra" SSTables in CF data folder, we are done.
// if (columnfamilyFiles.size() == 0)
// return;
//
// columnfamilyFiles.parallelStream().forEach(file -> logger.info("Forgotten file: {}
// found for CF: {}", file.getAbsolutePath(), columnfamilyDir.getName()));
//
// //TODO: The eventual plan is to move the forgotten files to a lost+found directory
// and clean the directory after 'x' amount of time. This behavior should be configurable.
// backupMetrics.incrementForgottenFiles(columnfamilyFiles.size());
// logger.warn("# of forgotten files: {} found for CF: {}", columnfamilyFiles.size(),
// columnfamilyDir.getName());
// } catch (Exception e) {
// //Eat the exception, if there, for any reason. This should not stop the snapshot
// for any reason.
// logger.error("Exception occurred while trying to find forgottenFile. Ignoring the
// error and continuing with remaining backup", e);
// }
// }

@Override
protected void addToRemotePath(String remotePath) {
snapshotRemotePaths.add(remotePath);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -18,7 +18,6 @@
package com.netflix.priam.backupv2;

import com.netflix.priam.backup.*;
import com.netflix.priam.config.IConfiguration;
import com.netflix.priam.utils.DateUtil;
import java.util.List;
import java.util.Optional;
Expand All @@ -33,15 +32,10 @@
*/
public class BackupValidator {
private static final Logger logger = LoggerFactory.getLogger(BackupVerification.class);
private final IBackupFileSystem fs;
private IMetaProxy metaProxy;

@Inject
public BackupValidator(
IConfiguration configuration,
IFileSystemContext backupFileSystemCtx,
@Named("v2") IMetaProxy metaProxy) {
fs = backupFileSystemCtx.getFileStrategy(configuration);
public BackupValidator(@Named("v2") IMetaProxy metaProxy) {
this.metaProxy = metaProxy;
}

Expand Down
Loading

0 comments on commit 43c7f1c

Please sign in to comment.