Skip to content

Commit

Permalink
[#7126] PITR: Restore deleted table
Browse files Browse the repository at this point in the history
Summary:
Adds the ability to restore the table that was previously deleted.
When the tablet that participates in the snapshot schedule is being deleted, it is marked as hidden instead of performing actual delete.
Such tablets reject reads and write, but could be restored to some point in time.

Cleanup for such tables should be implemented in follow-up diffs.

Test Plan: ybd --gtest_filter YbAdminSnapshotScheduleTest.SnapshotScheduleUndeleteTable

Reviewers: bogdan

Reviewed By: bogdan

Subscribers: rahuldesirazu, skedia, mbautin, ybase

Differential Revision: https://phabricator.dev.yugabyte.com/D11389
  • Loading branch information
spolitov committed May 5, 2021
1 parent 6aaa3bd commit 9fd73c7
Show file tree
Hide file tree
Showing 38 changed files with 758 additions and 334 deletions.
8 changes: 8 additions & 0 deletions ent/src/yb/master/async_snapshot_tasks.cc
Original file line number Diff line number Diff line change
Expand Up @@ -157,6 +157,7 @@ bool AsyncTabletSnapshotOp::SendRequest(int attempt) {
req.set_schema_version(schema_version_);
*req.mutable_schema() = schema_;
*req.mutable_indexes() = indexes_;
req.set_hide(hide_);
}
req.set_propagated_hybrid_time(master_->clock()->Now().ToUint64());

Expand Down Expand Up @@ -186,5 +187,12 @@ void AsyncTabletSnapshotOp::Finished(const Status& status) {
}
}

void AsyncTabletSnapshotOp::SetMetadata(const SysTablesEntryPB& pb) {
has_metadata_ = true;
schema_version_ = pb.version();
schema_ = pb.schema();
indexes_ = pb.indexes();
}

} // namespace master
} // namespace yb
9 changes: 2 additions & 7 deletions ent/src/yb/master/async_snapshot_tasks.h
Original file line number Diff line number Diff line change
Expand Up @@ -48,13 +48,7 @@ class AsyncTabletSnapshotOp : public enterprise::RetryingTSRpcTask {
snapshot_hybrid_time_ = value;
}

void SetMetadata(uint32_t schema_version, const SchemaPB& schema,
const google::protobuf::RepeatedPtrField<IndexInfoPB>& indexes) {
has_metadata_ = true;
schema_version_ = schema_version;
schema_ = schema;
indexes_ = indexes;
}
void SetMetadata(const SysTablesEntryPB& pb);

void SetCallback(TabletSnapshotOperationCallback callback) {
callback_ = std::move(callback);
Expand All @@ -79,6 +73,7 @@ class AsyncTabletSnapshotOp : public enterprise::RetryingTSRpcTask {
uint32_t schema_version_;
SchemaPB schema_;
google::protobuf::RepeatedPtrField<IndexInfoPB> indexes_;
bool hide_ = false;
};

} // namespace master
Expand Down
3 changes: 2 additions & 1 deletion ent/src/yb/master/catalog_manager.h
Original file line number Diff line number Diff line change
Expand Up @@ -282,7 +282,8 @@ class CatalogManager : public yb::master::CatalogManager, SnapshotCoordinatorCon

int64_t LeaderTerm() override;

Result<SnapshotSchedulesToTabletsMap> MakeSnapshotSchedulesToTabletsMap() override;
Result<SnapshotSchedulesToObjectIdsMap> MakeSnapshotSchedulesToObjectIdsMap(
SysRowEntry::Type type) override;

static void SetTabletSnapshotsState(SysSnapshotEntryPB::State state,
SysSnapshotEntryPB* snapshot_pb);
Expand Down
9 changes: 4 additions & 5 deletions ent/src/yb/master/catalog_manager_ent.cc
Original file line number Diff line number Diff line change
Expand Up @@ -1392,9 +1392,7 @@ void CatalogManager::SendRestoreTabletSnapshotRequest(
call->SetSnapshotHybridTime(restore_at);
}
if (send_metadata) {
auto lock = tablet->table()->LockForRead();
const auto& pb = lock->pb;
call->SetMetadata(pb.version(), pb.schema(), pb.indexes());
call->SetMetadata(tablet->table()->LockForRead()->pb);
}
call->SetCallback(std::move(callback));
tablet->table()->AddTask(call);
Expand Down Expand Up @@ -3404,8 +3402,9 @@ void CatalogManager::Started() {
snapshot_coordinator_.Start();
}

Result<SnapshotSchedulesToTabletsMap> CatalogManager::MakeSnapshotSchedulesToTabletsMap() {
return snapshot_coordinator_.MakeSnapshotSchedulesToTabletsMap();
Result<SnapshotSchedulesToObjectIdsMap> CatalogManager::MakeSnapshotSchedulesToObjectIdsMap(
SysRowEntry::Type type) {
return snapshot_coordinator_.MakeSnapshotSchedulesToObjectIdsMap(type);
}

void CatalogManager::SysCatalogLoaded(int64_t term) {
Expand Down
4 changes: 2 additions & 2 deletions src/yb/common/snapshot.h
Original file line number Diff line number Diff line change
Expand Up @@ -26,8 +26,8 @@ YB_STRONGLY_TYPED_UUID(TxnSnapshotId);
YB_STRONGLY_TYPED_UUID(TxnSnapshotRestorationId);
YB_STRONGLY_TYPED_UUID(SnapshotScheduleId);

using SnapshotSchedulesToTabletsMap =
std::unordered_map<SnapshotScheduleId, std::vector<TabletId>, SnapshotScheduleIdHash>;
using SnapshotSchedulesToObjectIdsMap =
std::unordered_map<SnapshotScheduleId, std::vector<std::string>, SnapshotScheduleIdHash>;

} // namespace yb

Expand Down
7 changes: 2 additions & 5 deletions src/yb/integration-tests/cassandra_cpp_driver-test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -2733,15 +2733,12 @@ TEST_F(CppCassandraDriverTest, BigQueryExpr) {
}

auto start = MonoTime::Now();
auto result = ASSERT_RESULT(session_.ExecuteWithResult(Format(
auto result = ASSERT_RESULT(session_.FetchValue<std::string>(Format(
"SELECT MAX(key) FROM $0", kTableName)));
auto finish = MonoTime::Now();
LOG(INFO) << "Time: " << finish - start;

auto iterator = result.CreateIterator();
ASSERT_TRUE(iterator.Next());
LOG(INFO) << "Result: " << iterator.Row().Value(0).ToString();
ASSERT_FALSE(iterator.Next());
LOG(INFO) << "Result: " << result;
}

class CppCassandraDriverSmallSoftLimitTest : public CppCassandraDriverTest {
Expand Down
9 changes: 9 additions & 0 deletions src/yb/integration-tests/cql_test_util.h
Original file line number Diff line number Diff line change
Expand Up @@ -257,6 +257,15 @@ class CassandraSession {
return ExecuteAndProcessOneRow(CassandraStatement(query), action);
}

template <class T>
Result<T> FetchValue(const std::string& query) {
T result = T();
RETURN_NOT_OK(ExecuteAndProcessOneRow(query, [&result](const CassandraRow& row) {
result = row.Value(0).As<T>();
}));
return result;
}

CHECKED_STATUS ExecuteBatch(const CassandraBatch& batch);

CassandraFuture SubmitBatch(const CassandraBatch& batch);
Expand Down
7 changes: 5 additions & 2 deletions src/yb/master/async_rpc_tasks.cc
Original file line number Diff line number Diff line change
Expand Up @@ -618,14 +618,14 @@ void AsyncDeleteReplica::HandleResponse(int attempt) {
if (table_) {
LOG_WITH_PREFIX(INFO)
<< "TS " << permanent_uuid_ << ": tablet " << tablet_id_
<< " (table " << table_->ToString() << ") successfully deleted";
<< " (table " << table_->ToString() << ") successfully done";
} else {
LOG_WITH_PREFIX(WARNING)
<< "TS " << permanent_uuid_ << ": tablet " << tablet_id_
<< " did not belong to a known table, but was successfully deleted";
}
TransitionToCompleteState();
VLOG_WITH_PREFIX(1) << "TS " << permanent_uuid_ << ": delete complete on tablet " << tablet_id_;
VLOG_WITH_PREFIX(1) << "TS " << permanent_uuid_ << ": complete on tablet " << tablet_id_;
}
}

Expand All @@ -635,6 +635,9 @@ bool AsyncDeleteReplica::SendRequest(int attempt) {
req.set_tablet_id(tablet_id_);
req.set_reason(reason_);
req.set_delete_type(delete_type_);
if (hide_only_) {
req.set_hide_only(hide_only_);
}
if (cas_config_opid_index_less_or_equal_) {
req.set_cas_config_opid_index_less_or_equal(*cas_config_opid_index_less_or_equal_);
}
Expand Down
8 changes: 7 additions & 1 deletion src/yb/master/async_rpc_tasks.h
Original file line number Diff line number Diff line change
Expand Up @@ -368,7 +368,12 @@ class AsyncDeleteReplica : public RetrySpecificTSRpcTask {
std::string type_name() const override { return "Delete Tablet"; }

std::string description() const override {
return "Delete Tablet RPC for " + tablet_id_ + " on TS=" + permanent_uuid_;
return Format("$0 Tablet RPC for $1 on TS=$2",
hide_only_ ? "Hide" : "Delete", tablet_id_, permanent_uuid_);
}

void set_hide_only(bool value) {
hide_only_ = value;
}

protected:
Expand All @@ -383,6 +388,7 @@ class AsyncDeleteReplica : public RetrySpecificTSRpcTask {
const boost::optional<int64_t> cas_config_opid_index_less_or_equal_;
const std::string reason_;
tserver::DeleteTabletResponsePB resp_;
bool hide_only_ = false;
};

// Send the "Alter Table" with the latest table schema to the leader replica
Expand Down
3 changes: 2 additions & 1 deletion src/yb/master/catalog_entity_info.cc
Original file line number Diff line number Diff line change
Expand Up @@ -396,7 +396,8 @@ bool TableInfo::IsAlterInProgress(uint32_t version) const {
bool TableInfo::AreAllTabletsDeleted() const {
shared_lock<decltype(lock_)> l(lock_);
for (const TableInfo::TabletInfoMap::value_type& e : tablet_map_) {
if (!e.second->LockForRead()->is_deleted()) {
auto tablet_lock = e.second->LockForRead();
if (!tablet_lock->is_deleted() && !tablet_lock->pb.hidden()) {
return false;
}
}
Expand Down
Loading

0 comments on commit 9fd73c7

Please sign in to comment.