Skip to content

Commit

Permalink
[BACKPORT 2024.1][#22876][#22773] CDCSDK: Add new yb-admin command to…
Browse files Browse the repository at this point in the history
… remove user table from CDCSDK stream

Summary:
**Backport Descrption:**
Faced minor conflict in yb-admin_cli.cc since some xcluster commands are not backported.

**Original Description:**
Original commit: 7c99ff9 / D35870
This diff introduces three new yb-admin commands required to remove a **user table** from a CDCSDK stream.
**`NOTE: All three commands are only meant to be used on CDC streams that are not associated with a replication slot.`**

**Command-1**: yb-admin command to disable dynamic table addition in a CDC stream. Only works when the new auto flag `enable_cdcsdk_dynamic_tables_disable_option` is set to true. **Note, post execution of this command, no dynamic tables (user/non-user) will get added to CDC stream. Additionally, there is no option to re-enable dynamic table addition for the stream.**
```
yb-admin \
    -master_addresses <master-addresses> \
    disable_dynamic_table_addition_in_change_data_stream <stream_id>
```
The command works with a single stream_id.

**Command-2**: yb-admin command to remove only a particular **user** table from the CDC stream metadata as well as update the checkpoint for corresponding state table entries to OpId max. Since, the checkpoint is set to max, these entries will be later deleted from the cdc state table by a separate thread (UpdatePeersAndMetrics).

```
yb-admin \
    -master_addresses <master-addresses> \
    remove_user_table_from_change_data_stream <stream_id> <table_id>
```
The command works with a single stream_id & table_id.

**Command-3**: yb-admin command to validate cdc state table entries for a particular stream. As part of validation, if the table of any cdc state table entry is not present in the CDC stream metadata, then checkpoint of such entries will be updated to OpID max, and they'll be later deleted by a separate thread (UpdatePeersAndMetrics).

```
yb-admin \
    -master_addresses <master-addresses> \
    validate_and_sync_cdc_state_table_entries_on_change_data_stream <stream_id>
```
The command works with a single stream_id.

**Advisory for command-usage:**
General guidelines that need to strictly followed while executing these commands:
  - Ensure no DDLs are performed before/after 15 mins of executing these commands.

These yb-admin commands are meant to be used when a user is only interested on polling from subset of tables in the namespace. Therefore, the user can remove the extra tables from CDC stream that are not supposed to be polled. To achieve this, user needs to first execute Command-1, followed by command-2 & command-3.

Example:
Starting state: 5 user tables (t1 to t5) in the CDC stream including 4 extra tables that are not polled (t1,t2,t3,t4) + 2 indexes (i1,i2)
Target state: Only t5 + 2 indexes (i1,i2) should be present in CDC stream.

To reach the target state, we need to remove 4 user tables (t1-t4) from stream metadata & their state entries

**Perform the following steps to remove user tables from the CDC stream:**

  # Firstly, disable dynamic table addition using command-1.
  # Confirm that dynamic table addition is disabled by running `list_change_data_streams` yb-admin command. The output for that stream would contain the string `cdcsdk_disable_dynamic_table_addition: true`
  # Remove the table from stream metadata & update its state table entries using command-2.
  # Confirm that the table is removed from stream metadata by re-running `list_change_data_streams` command.
  # Based on when the user reads the cdc state table (via cqlsh), the state table entries corresponding to this table would have been either updated to checkpoint max or may be removed. Note, State table entries deletion might take some time as it will be done in a separate thread.
  # Repeat step 3-5 for all user tables that needs to be removed.
  # At the end, once all extra user tables are removed from a stream, execute command-3 as a sanity check to get rid of any cdc state entries that might still be hanging around in state table but the corresponding table has been removed from stream metadata. One scenario where cdc state table entries might be present even after table is removed, is when a tablet splits while table was being removed from stream metadata. In this case, the children tablet entries will get added to cdc state table and so they'll get removed when command-3 is executed.

**Working**:
Command-1 internally calls //DisableDynamicTableAdditionOnCDCSDKStream// RPC that will set the optional field `cdcsdk_disable_dynamic_table_addition` in stream metadata to true. This will prevent any tables, that are not yet part of the CDC stream, to get added to the CDC stream.

Command-2 internally calls //RemoveUserTableFromCDCSDKStream// RPC that performs the following:
  # Update the checkpoint of tablet entries for the given table in the CDC state table to `OpId::Max()`. This is done to release the retention barriers on these tables and allow the deletion of the state table entry  by UpdatePeersAndMetrics.
  # Remove the table from CDC stream metadata, //cdcsdk_tables_to_stream_map_// and persist the updated metadata in sys catalog.

Command-3 internally calls //ValidateAndSyncCDCStateEntriesForCDCSDKStream// RPC that updates checkpoint to max for cdc state table entries whose table is not found in the CDC stream metadata.

**Upgrade/Rollback safety:**
//cdcsdk_disable_dynamic_table_addition// - added a new optional field in existing protos SysCDCStreamEntryPB, CDCStreamInfoPB. This field is protected and will only be read when the new auto flag `cdcsdk_enable_dynamic_tables_disable_option` is set.

Introduced request, response proto for new RPCs:

  - DisableDynamicTableAdditionOnCDCSDKStream - DisableDynamicTableAdditionOnCDCSDKStreamRequestPB,  DisableDynamicTableAdditionOnCDCSDKStreamResponsePB
  - RemoveUserTableFromCDCSDKStream - RemoveUserTableFromCDCSDKStreamRequestPB,  RemoveUserTableFromCDCSDKStreamResponsePB
  - ValidateAndSyncCDCStateEntriesForCDCSDKStream - ValidateAndSyncCDCStateEntriesForCDCSDKStreamRequestPB, ValidateAndSyncCDCStateEntriesForCDCSDKStreamResponsePB
Jira: DB-11778, DB-11676

Test Plan:
Jenkins: urgent
./yb_build.sh --cxx-test integration-tests_cdcsdk_ysql-test --gtest_filter CDCSDKYsqlTest.TestDisableOfDynamicTableAdditionOnNonConsistentSnapshotStream
./yb_build.sh --cxx-test integration-tests_cdcsdk_ysql-test --gtest_filter CDCSDKYsqlTest.TestDisableOfDynamicTableAdditionOnConsistentSnapshotStream

./yb_build.sh --cxx-test integration-tests_cdcsdk_ysql-test --gtest_filter CDCSDKYsqlTest.TestUserTableRemovalFromNonConsistentSnapshotCDCStream
./yb_build.sh --cxx-test integration-tests_cdcsdk_ysql-test --gtest_filter CDCSDKYsqlTest.TestUserTableRemovalFromConsistentSnapshotCDCStream

./yb_build.sh --cxx-test integration-tests_cdcsdk_ysql-test --gtest_filter CDCSDKYsqlTest.TestValidationAndSyncOfCDCStateEntriesAfterUserTableRemovalOnNonConsistentSnapshotStream
./yb_build.sh --cxx-test integration-tests_cdcsdk_ysql-test --gtest_filter CDCSDKYsqlTest.TestValidationAndSyncOfCDCStateEntriesAfterUserTableRemovalOnConsistentSnapshotStream

Reviewers: skumar, asrinivasan, stiwary

Reviewed By: asrinivasan, stiwary

Subscribers: ybase, ycdcxcluster

Tags: #jenkins-ready

Differential Revision: https://phorge.dev.yugabyte.com/D36129
  • Loading branch information
siddharth2411 committed Jun 26, 2024
1 parent c2c8787 commit 1040594
Show file tree
Hide file tree
Showing 13 changed files with 1,015 additions and 4 deletions.
356 changes: 356 additions & 0 deletions src/yb/integration-tests/cdcsdk_ysql-test.cc

Large diffs are not rendered by default.

48 changes: 48 additions & 0 deletions src/yb/integration-tests/cdcsdk_ysql_test_base.cc
Original file line number Diff line number Diff line change
Expand Up @@ -4491,5 +4491,53 @@ Result<string> CDCSDKYsqlTest::GetUniverseId(PostgresMiniCluster* cluster) {
return oss.str();
}

Status CDCSDKYsqlTest::ExecuteYBAdminCommand(
const std::string& command_name, const std::vector<string>& command_args) {
string tool_path = GetToolPath("../bin", "yb-admin");
vector<string> argv;
argv.push_back(tool_path);
argv.push_back("--master_addresses");
argv.push_back(AsString(test_cluster_.mini_cluster_->GetMasterAddresses()));
argv.push_back(command_name);
for (const auto& command_arg : command_args) {
argv.push_back(command_arg);
}

RETURN_NOT_OK(Subprocess::Call(argv));

return Status::OK();
}

Status CDCSDKYsqlTest::DisableDynamicTableAdditionOnCDCSDKStream(
const xrepl::StreamId& stream_id) {
std::string yb_admin_command = "disable_dynamic_table_addition_on_change_data_stream";
vector<string> command_args;
command_args.push_back(stream_id.ToString());
RETURN_NOT_OK(ExecuteYBAdminCommand(yb_admin_command, command_args));
return Status::OK();
}

Status CDCSDKYsqlTest::RemoveUserTableFromCDCSDKStream(
const xrepl::StreamId& stream_id, const TableId& table_id) {
std::string yb_admin_command = "remove_user_table_from_change_data_stream";
vector<string> command_args;
command_args.push_back(stream_id.ToString());
command_args.push_back(table_id);
RETURN_NOT_OK(ExecuteYBAdminCommand(yb_admin_command, command_args));

return Status::OK();
}

Status CDCSDKYsqlTest::ValidateAndSyncCDCStateEntriesForCDCSDKStream(
const xrepl::StreamId& stream_id) {
std::string yb_admin_command =
"validate_and_sync_cdc_state_table_entries_on_change_data_stream";
vector<string> command_args;
command_args.push_back(stream_id.ToString());
RETURN_NOT_OK(ExecuteYBAdminCommand(yb_admin_command, command_args));

return Status::OK();
}

} // namespace cdc
} // namespace yb
18 changes: 18 additions & 0 deletions src/yb/integration-tests/cdcsdk_ysql_test_base.h
Original file line number Diff line number Diff line change
Expand Up @@ -117,6 +117,8 @@ DECLARE_uint64(TEST_cdcsdk_publication_list_refresh_interval_micros);
DECLARE_bool(cdcsdk_enable_dynamic_table_support);
DECLARE_bool(enable_cdcsdk_setting_get_changes_response_byte_limit);
DECLARE_uint64(cdcsdk_vwal_getchanges_resp_max_size_bytes);
DECLARE_bool(cdcsdk_enable_dynamic_tables_disable_option);
DECLARE_bool(TEST_cdcsdk_skip_updating_cdc_state_entries_on_table_removal);

namespace yb {

Expand Down Expand Up @@ -781,6 +783,22 @@ class CDCSDKYsqlTest : public CDCSDKTestBase {
std::string GetPubRefreshTimesString(vector<uint64_t> pub_refresh_times);

void TestNonUserTableShouldNotGetAddedToCDCStream (bool create_consistent_snapshot_stream);

Status ExecuteYBAdminCommand(
const std::string& command_name, const std::vector<string>& command_args);

Status DisableDynamicTableAdditionOnCDCSDKStream(const xrepl::StreamId& stream_id);

void TestDisableOfDynamicTableAdditionOnCDCStream(bool use_consistent_snapshot_stream);

Status RemoveUserTableFromCDCSDKStream(const xrepl::StreamId& stream_id, const TableId& table_id);

void TestUserTableRemovalFromCDCStream(bool use_consistent_snapshot_stream);

Status ValidateAndSyncCDCStateEntriesForCDCSDKStream(const xrepl::StreamId& stream_id);

void TestValidationAndSyncOfCDCStateEntriesAfterUserTableRemoval(
bool use_consistent_snapshot_stream);
};

} // namespace cdc
Expand Down
11 changes: 11 additions & 0 deletions src/yb/master/catalog_entity_info.cc
Original file line number Diff line number Diff line change
Expand Up @@ -60,6 +60,7 @@ using std::string;
using strings::Substitute;

DECLARE_int32(tserver_unresponsive_timeout_ms);
DECLARE_bool(cdcsdk_enable_dynamic_tables_disable_option);

DEFINE_RUNTIME_AUTO_bool(
use_parent_table_id_field, kLocalPersisted, false, true,
Expand Down Expand Up @@ -1268,6 +1269,16 @@ CDCStreamInfo::GetReplicaIdentityMap() const {
return l->pb.replica_identity_map();
}

bool CDCStreamInfo::IsDynamicTableAdditionDisabled() const {
if (!FLAGS_cdcsdk_enable_dynamic_tables_disable_option) {
return false;
}

auto l = LockForRead();
return l->pb.has_cdcsdk_disable_dynamic_table_addition() &&
l->pb.cdcsdk_disable_dynamic_table_addition();
}

std::string CDCStreamInfo::ToString() const {
auto l = LockForRead();
if (l->pb.has_namespace_id()) {
Expand Down
2 changes: 2 additions & 0 deletions src/yb/master/catalog_entity_info.h
Original file line number Diff line number Diff line change
Expand Up @@ -1171,6 +1171,8 @@ class CDCStreamInfo : public RefCountedThreadSafe<CDCStreamInfo>,

const google::protobuf::Map<::std::string, ::yb::PgReplicaIdentity> GetReplicaIdentityMap() const;

bool IsDynamicTableAdditionDisabled() const;

std::string ToString() const override;

bool IsXClusterStream() const;
Expand Down
5 changes: 5 additions & 0 deletions src/yb/master/catalog_entity_info.proto
Original file line number Diff line number Diff line change
Expand Up @@ -535,6 +535,11 @@ message SysCDCStreamEntryPB {
map<string, PgReplicaIdentity> replica_identity_map = 9;

optional string cdcsdk_ysql_replication_slot_plugin_name = 10;

// Dynamic tables are the tables which are created after the creation of the stream.
// This field controls if dynamic tables should automatically be added to the CDC stream or not.
// If set to true, dynamic table wont get added to the CDC stream.
optional bool cdcsdk_disable_dynamic_table_addition = 11;
}


Expand Down
23 changes: 22 additions & 1 deletion src/yb/master/catalog_manager.h
Original file line number Diff line number Diff line change
Expand Up @@ -45,6 +45,7 @@
#include <gtest/internal/gtest-internal.h>

#include "yb/cdc/cdc_service.pb.h"
#include "yb/cdc/cdc_state_table.h"
#include "yb/cdc/xcluster_types.h"
#include "yb/common/constants.h"
#include "yb/common/entity_ids.h"
Expand Down Expand Up @@ -1342,6 +1343,18 @@ class CatalogManager : public tserver::TabletPeerLookupIf,
YsqlBackfillReplicationSlotNameToCDCSDKStreamResponsePB* resp,
rpc::RpcContext* rpc);

Status DisableDynamicTableAdditionOnCDCSDKStream(
const DisableDynamicTableAdditionOnCDCSDKStreamRequestPB* req,
DisableDynamicTableAdditionOnCDCSDKStreamResponsePB* resp, rpc::RpcContext* rpc);

Status RemoveUserTableFromCDCSDKStream(
const RemoveUserTableFromCDCSDKStreamRequestPB* req,
RemoveUserTableFromCDCSDKStreamResponsePB* resp, rpc::RpcContext* rpc);

Status ValidateAndSyncCDCStateEntriesForCDCSDKStream(
const ValidateAndSyncCDCStateEntriesForCDCSDKStreamRequestPB* req,
ValidateAndSyncCDCStateEntriesForCDCSDKStreamResponsePB* resp, rpc::RpcContext* rpc);

// Query if Bootstrapping is required for a CDC stream (e.g. Are we missing logs).
Status IsBootstrapRequired(
const IsBootstrapRequiredRequestPB* req,
Expand Down Expand Up @@ -1472,7 +1485,7 @@ class CatalogManager : public tserver::TabletPeerLookupIf,
// Find all CDCSDK streams which do not have metadata for the newly added tables.
Status FindCDCSDKStreamsForAddedTables(TableStreamIdsMap* table_to_unprocessed_streams_map);

bool CanTableBeAddedToCDCSDKStream(
bool IsTableEligibleForCDCSDKStream(
const TableInfoPtr& table_info, const Schema& schema) const REQUIRES_SHARED(mutex_);

// This method compares all tables in the namespace to all the tables added to a CDCSDK stream,
Expand Down Expand Up @@ -3191,6 +3204,14 @@ class CatalogManager : public tserver::TabletPeerLookupIf,
const TabletInfo& tablet, const ScheduleMinRestoreTime& schedule_to_min_restore_time)
EXCLUDES(mutex_);

Result<std::vector<cdc::CDCStateTableEntry>> UpdateCheckpointForTabletEntriesInCDCState(
const xrepl::StreamId& stream_id,
const std::unordered_set<TableId>& tables_in_stream_metadata,
const TableId& table_to_be_removed = "");

Status RemoveTableFromCDCStreamMetadataAndMaps(
const CDCStreamInfoPtr stream, const TableId table_id);

// Should be bumped up when tablet locations are changed.
std::atomic<uintptr_t> tablet_locations_version_{0};

Expand Down
42 changes: 42 additions & 0 deletions src/yb/master/master_replication.proto
Original file line number Diff line number Diff line change
Expand Up @@ -42,6 +42,11 @@ message CDCStreamInfoPB {
map<string, PgReplicaIdentity> replica_identity_map = 10;

optional string cdcsdk_ysql_replication_slot_plugin_name = 11;

// Dynamic tables are the tables which are created after the creation of the stream.
// This field controls if dynamic tables should automatically be added to the CDC stream or not.
// If set to true, dynamic table wont get added to the CDC stream.
optional bool cdcsdk_disable_dynamic_table_addition = 12;
}

message ValidateReplicationInfoRequestPB {
Expand Down Expand Up @@ -755,6 +760,32 @@ message GetUniverseReplicationInfoResponsePB {
repeated DbScopedInfoPB db_scoped_infos = 5;
}

message DisableDynamicTableAdditionOnCDCSDKStreamRequestPB {
optional string stream_id = 1;
}

message DisableDynamicTableAdditionOnCDCSDKStreamResponsePB {
optional MasterErrorPB error = 1;
}

message RemoveUserTableFromCDCSDKStreamRequestPB {
optional string stream_id = 1;
optional string table_id = 2;
}

message RemoveUserTableFromCDCSDKStreamResponsePB {
optional MasterErrorPB error = 1;
}

message ValidateAndSyncCDCStateEntriesForCDCSDKStreamRequestPB {
optional string stream_id = 1;
}

message ValidateAndSyncCDCStateEntriesForCDCSDKStreamResponsePB {
optional MasterErrorPB error = 1;
repeated string updated_tablet_entries = 2;
}

service MasterReplication {
option (yb.rpc.custom_service_name) = "yb.master.MasterService";

Expand Down Expand Up @@ -871,4 +902,15 @@ service MasterReplication {
returns (AddNamespaceToXClusterReplicationResponsePB);
rpc IsAlterXClusterReplicationDone(IsAlterXClusterReplicationDoneRequestPB)
returns (IsAlterXClusterReplicationDoneResponsePB);

// Introduced for bug (#22876, #22773)
rpc DisableDynamicTableAdditionOnCDCSDKStream (DisableDynamicTableAdditionOnCDCSDKStreamRequestPB)
returns (DisableDynamicTableAdditionOnCDCSDKStreamResponsePB);
// Introduced for bug (#22876, #22773)
rpc RemoveUserTableFromCDCSDKStream (RemoveUserTableFromCDCSDKStreamRequestPB)
returns (RemoveUserTableFromCDCSDKStreamResponsePB);
// Introduced for bug (#22876, #22773)
rpc ValidateAndSyncCDCStateEntriesForCDCSDKStream(
ValidateAndSyncCDCStateEntriesForCDCSDKStreamRequestPB)
returns (ValidateAndSyncCDCStateEntriesForCDCSDKStreamResponsePB);
}
3 changes: 3 additions & 0 deletions src/yb/master/master_replication_service.cc
Original file line number Diff line number Diff line change
Expand Up @@ -57,6 +57,9 @@ class MasterReplicationServiceImpl : public MasterServiceBase, public MasterRepl
(ChangeXClusterRole)
(BootstrapProducer)
(YsqlBackfillReplicationSlotNameToCDCSDKStream)
(DisableDynamicTableAdditionOnCDCSDKStream)
(RemoveUserTableFromCDCSDKStream)
(ValidateAndSyncCDCStateEntriesForCDCSDKStream)
)

MASTER_SERVICE_IMPL_ON_LEADER_WITH_LOCK(
Expand Down
Loading

0 comments on commit 1040594

Please sign in to comment.