Skip to content
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -1403,9 +1403,7 @@ public static void restoreFromSnapshot(Connection conn) throws IOException {
try (Admin admin = conn.getAdmin()) {
String snapshotName = BackupSystemTable.getSnapshotName(conf);
if (snapshotExists(admin, snapshotName)) {
admin.disableTable(BackupSystemTable.getTableName(conf));
admin.restoreSnapshot(snapshotName);
admin.enableTable(BackupSystemTable.getTableName(conf));
admin.restoreBackupSystemTable(snapshotName);
LOG.debug("Done restoring backup system table");
} else {
// Snapshot does not exists, i.e completeBackup failed after
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,84 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hbase.backup.master;

import static org.junit.Assert.assertEquals;

import java.util.HashMap;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;
import org.apache.hadoop.hbase.HBaseTestingUtility;
import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.backup.impl.BackupSystemTable;
import org.apache.hadoop.hbase.client.Admin;
import org.apache.hadoop.hbase.testclassification.MasterTests;
import org.apache.hadoop.hbase.testclassification.MediumTests;
import org.junit.AfterClass;
import org.junit.BeforeClass;
import org.junit.Test;
import org.junit.experimental.categories.Category;

@Category({ MasterTests.class, MediumTests.class })
public class TestRestoreBackupSystemTable {
private static final String BACKUP_ROOT = "root";
private static final HBaseTestingUtility UTIL = new HBaseTestingUtility();

@BeforeClass
public static void setUp() throws Exception {
UTIL.startMiniCluster();
}

@Test
public void itRestoresFromSnapshot() throws Exception {
BackupSystemTable table = new BackupSystemTable(UTIL.getConnection());
Set<TableName> tables = new HashSet<>();

tables.add(TableName.valueOf("test1"));
tables.add(TableName.valueOf("test2"));
tables.add(TableName.valueOf("test3"));

Map<String, Long> rsTimestampMap = new HashMap<>();
rsTimestampMap.put("rs1:100", 100L);
rsTimestampMap.put("rs2:100", 101L);
rsTimestampMap.put("rs3:100", 103L);

table.writeRegionServerLogTimestamp(tables, rsTimestampMap, BACKUP_ROOT);
BackupSystemTable.snapshot(UTIL.getConnection());

Admin admin = UTIL.getAdmin();
TableName backupSystemTn = BackupSystemTable.getTableName(UTIL.getConfiguration());
admin.disableTable(backupSystemTn);
admin.truncateTable(backupSystemTn, true);

BackupSystemTable.restoreFromSnapshot(UTIL.getConnection());
Map<TableName, Map<String, Long>> results = table.readLogTimestampMap(BACKUP_ROOT);

assertEquals(results.size(), tables.size());

for (TableName tableName : tables) {
Map<String, Long> resultMap = results.get(tableName);
assertEquals(resultMap, rsTimestampMap);
}
}

@AfterClass
public static void tearDown() throws Exception {
UTIL.shutdownMiniCluster();
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -3356,4 +3356,15 @@ List<LogEntry> getLogEntries(Set<ServerName> serverNames, String logType, Server
* Get the list of cached files
*/
List<String> getCachedFilesList(ServerName serverName) throws IOException;

@InterfaceAudience.Private
default void restoreBackupSystemTable(String snapshotName) throws IOException {
SnapshotDescription snapshot =
listSnapshots().stream().filter(s -> s.getName().equals(snapshotName)).findFirst()
.orElseThrow(() -> new IOException("Snapshot " + snapshotName + " not found"));
TableName tn = snapshot.getTableName();
disableTable(tn);
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This is the same behavior as the logic we had in the BackupSystemTable that we're replacing. Does it makse sense to include this behavior? Or should this method do nothing by default?

restoreSnapshot(snapshotName);
enableTable(tn);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -1721,4 +1721,7 @@ CompletableFuture<List<LogEntry>> getLogEntries(Set<ServerName> serverNames, Str
* Get the list of cached files
*/
CompletableFuture<List<String>> getCachedFilesList(ServerName serverName);

@InterfaceAudience.Private
CompletableFuture<Void> restoreBackupSystemTable(String snapshotName);
}
Original file line number Diff line number Diff line change
Expand Up @@ -921,4 +921,9 @@ public CompletableFuture<Void> flushMasterStore() {
public CompletableFuture<List<String>> getCachedFilesList(ServerName serverName) {
return wrap(rawAdmin.getCachedFilesList(serverName));
}

@Override
public CompletableFuture<Void> restoreBackupSystemTable(String snapshotName) {
return wrap(rawAdmin.restoreBackupSystemTable(snapshotName));
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -2082,6 +2082,14 @@ public FlushMasterStoreResponse flushMasterStore(RpcController controller,
return stub.flushMasterStore(controller, request);
}

@Override
public MasterProtos.RestoreBackupSystemTableResponse restoreBackupSystemTable(
RpcController rpcController,
MasterProtos.RestoreBackupSystemTableRequest restoreBackupSystemTableRequest)
throws ServiceException {
return stub.restoreBackupSystemTable(rpcController, restoreBackupSystemTableRequest);
}

@Override
public ReplicationPeerModificationSwitchResponse replicationPeerModificationSwitch(
RpcController controller, ReplicationPeerModificationSwitchRequest request)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2095,6 +2095,22 @@ public List<String> getCachedFilesList(ServerName serverName) throws IOException
this.connection.getAdmin(serverName));
}

@Override
public void restoreBackupSystemTable(String snapshotName) throws IOException {
long pid =
executeCallable(new MasterCallable<Long>(getConnection(), getRpcControllerFactory()) {
@Override
protected Long rpcCall() throws Exception {
return master.restoreBackupSystemTable(getRpcController(),
MasterProtos.RestoreBackupSystemTableRequest.newBuilder().setSnapshotName(snapshotName)
.build())
.getProcId();
}
});
ProcedureFuture<Void> future = new ProcedureFuture<>(this, pid);
get(future, getProcedureTimeout, TimeUnit.MILLISECONDS);
}

private MasterCallable<MasterProtos.TruncateRegionResponse>
getTruncateRegionCallable(TableName tableName, RegionInfo hri) {
return new MasterCallable<MasterProtos.TruncateRegionResponse>(getConnection(),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2716,6 +2716,19 @@ void onError(Throwable error) {
}
}

private static class RestoreBackupSystemTableProcedureBiConsumer extends ProcedureBiConsumer {

@Override
void onFinished() {
LOG.info("RestoreBackupSystemTableProcedure completed");
}

@Override
void onError(Throwable error) {
LOG.info("RestoreBackupSystemTableProcedure failed with {}", error.getMessage());
}
}

private static class CreateTableProcedureBiConsumer extends TableProcedureBiConsumer {

CreateTableProcedureBiConsumer(TableName tableName) {
Expand Down Expand Up @@ -4299,4 +4312,16 @@ List<String>> adminCall(controller, stub, request.build(),
resp -> resp.getCachedFilesList()))
.serverName(serverName).call();
}

@Override
public CompletableFuture<Void> restoreBackupSystemTable(String snapshotName) {
MasterProtos.RestoreBackupSystemTableRequest request =
MasterProtos.RestoreBackupSystemTableRequest.newBuilder().setSnapshotName(snapshotName)
.build();
return this.<MasterProtos.RestoreBackupSystemTableRequest,
MasterProtos.RestoreBackupSystemTableResponse> procedureCall(request,
MasterService.Interface::restoreBackupSystemTable,
MasterProtos.RestoreBackupSystemTableResponse::getProcId,
new RestoreBackupSystemTableProcedureBiConsumer());
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -796,6 +796,13 @@ public FlushMasterStoreResponse flushMasterStore(RpcController controller,
return stub.flushMasterStore(controller, request);
}

@Override
public MasterProtos.RestoreBackupSystemTableResponse restoreBackupSystemTable(
RpcController controller, MasterProtos.RestoreBackupSystemTableRequest request)
throws ServiceException {
return stub.restoreBackupSystemTable(controller, request);
}

@Override
public ReplicationPeerModificationSwitchResponse replicationPeerModificationSwitch(
RpcController controller, ReplicationPeerModificationSwitchRequest request)
Expand Down
10 changes: 10 additions & 0 deletions hbase-protocol-shaded/src/main/protobuf/Master.proto
Original file line number Diff line number Diff line change
Expand Up @@ -1228,6 +1228,9 @@ service MasterService {

rpc FlushMasterStore(FlushMasterStoreRequest)
returns(FlushMasterStoreResponse);

rpc RestoreBackupSystemTable(RestoreBackupSystemTableRequest)
returns(RestoreBackupSystemTableResponse);
}

// HBCK Service definitions.
Expand Down Expand Up @@ -1313,6 +1316,13 @@ message FixMetaRequest {}

message FixMetaResponse {}

message RestoreBackupSystemTableRequest {
required string snapshot_name = 1;
}
message RestoreBackupSystemTableResponse {
optional uint64 proc_id = 1;
}

service HbckService {
/** Update state of the table in meta only*/
rpc SetTableStateInMeta(SetTableStateInMetaRequest)
Expand Down
7 changes: 7 additions & 0 deletions hbase-protocol-shaded/src/main/protobuf/MasterProcedure.proto
Original file line number Diff line number Diff line change
Expand Up @@ -743,3 +743,10 @@ message ReloadQuotasProcedureStateData {
required ServerName target_server = 1;
optional ForeignExceptionMessage error = 2;
}

enum RestoreBackupSystemTableState {
RESTORE_BACKUP_SYSTEM_TABLE_PREPARE = 1;
RESTORE_BACKUP_SYSTEM_TABLE_DISABLE = 2;
RESTORE_BACKUP_SYSTEM_TABLE_RESTORE = 3;
RESTORE_BACKUP_SYSTEM_TABLE_ENABLE = 4;
}
Original file line number Diff line number Diff line change
Expand Up @@ -84,6 +84,7 @@
import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
import org.apache.hadoop.hbase.master.procedure.MasterProcedureUtil;
import org.apache.hadoop.hbase.master.procedure.MasterProcedureUtil.NonceProcedureRunnable;
import org.apache.hadoop.hbase.master.procedure.RestoreBackupSystemTableProcedure;
import org.apache.hadoop.hbase.master.procedure.ServerCrashProcedure;
import org.apache.hadoop.hbase.master.replication.AbstractPeerProcedure;
import org.apache.hadoop.hbase.mob.MobUtils;
Expand Down Expand Up @@ -3168,4 +3169,22 @@ public FlushTableResponse flushTable(RpcController controller, FlushTableRequest
throw new ServiceException(ioe);
}
}

@Override
public MasterProtos.RestoreBackupSystemTableResponse restoreBackupSystemTable(
RpcController rpcController,
MasterProtos.RestoreBackupSystemTableRequest restoreBackupSystemTableRequest)
throws ServiceException {
try {
String snapshotName = restoreBackupSystemTableRequest.getSnapshotName();
SnapshotDescription snapshot = master.snapshotManager.getCompletedSnapshots().stream()
.filter(s -> s.getName().equals(snapshotName)).findFirst().orElseThrow(
() -> new ServiceException(String.format("Snapshot %s not found", snapshotName)));
long pid = master.getMasterProcedureExecutor()
.submitProcedure(new RestoreBackupSystemTableProcedure(snapshot));
return MasterProtos.RestoreBackupSystemTableResponse.newBuilder().setProcId(pid).build();
} catch (IOException e) {
throw new ServiceException(e);
}
}
}
Loading