tablets, mv: fix on_internal_error on write to base table

This situation before this patch is that when tablets are enabled for
a keyspace, we can create a materialized view but later any write to
the base table fails with an on_internal_error(), saying that:

     "Tried to obtain per-keyspace effective replication map of test
      but it's per-table."

Indeed, with tablets, the replication is different for each table - it's
not the same for the entire keyspace.

So this patch changes the view update code to take the replication
map from the specific base table, not the keyspace.

This is good enough to get materialized-views reads and writes working
in a simple single-node case, as the included test demonstrates (the
test fails with on_internal_error() before this patch, and passes
afterwards).

But this fix is not perfect - the base-view pairing code really needs
to consider not only the base table's replication map, but also the
view table's replication map - as those can be different. We'll fix
this remaining problem as a followup in a separate patch - it will
require a substantially more elaborate test to reproduce the need
for the different mapping and to verify that fix.

Fixes #16209.

Signed-off-by: Nadav Har'El <nyh@scylladb.com>

Closes scylladb/scylladb#16211
This commit is contained in:
Nadav Har'El
2023-11-29 00:39:32 +02:00
committed by Tomasz Grabiec
parent cd732b1364
commit 62f89d49e5
2 changed files with 24 additions and 4 deletions

View File

@@ -1546,7 +1546,7 @@ bool needs_static_row(const mutation_partition& mp, const std::vector<view_and_b
// If the assumption that the given base token belongs to this replica // If the assumption that the given base token belongs to this replica
// does not hold, we return an empty optional. // does not hold, we return an empty optional.
static std::optional<gms::inet_address> static std::optional<gms::inet_address>
get_view_natural_endpoint(const locator::vnode_effective_replication_map_ptr& erm, bool network_topology, get_view_natural_endpoint(const locator::effective_replication_map_ptr& erm, bool network_topology,
const dht::token& base_token, const dht::token& view_token) { const dht::token& base_token, const dht::token& view_token) {
auto& topology = erm->get_token_metadata_ptr()->get_topology(); auto& topology = erm->get_token_metadata_ptr()->get_topology();
auto my_address = utils::fb_utilities::get_broadcast_address(); auto my_address = utils::fb_utilities::get_broadcast_address();
@@ -1630,9 +1630,8 @@ future<> view_update_generator::mutate_MV(
co_await max_concurrent_for_each(view_updates, max_concurrent_updates, co_await max_concurrent_for_each(view_updates, max_concurrent_updates,
[this, base_token, &stats, &cf_stats, tr_state, &pending_view_updates, allow_hints, wait_for_all] (frozen_mutation_and_schema mut) mutable -> future<> { [this, base_token, &stats, &cf_stats, tr_state, &pending_view_updates, allow_hints, wait_for_all] (frozen_mutation_and_schema mut) mutable -> future<> {
auto view_token = dht::get_token(*mut.s, mut.fm.key()); auto view_token = dht::get_token(*mut.s, mut.fm.key());
auto& keyspace_name = mut.s->ks_name(); auto ermp = mut.s->table().get_effective_replication_map();
auto& ks = _proxy.local().local_db().find_keyspace(keyspace_name); auto& ks = _proxy.local().local_db().find_keyspace(mut.s->ks_name());
auto ermp = ks.get_effective_replication_map();
bool network_topology = dynamic_cast<const locator::network_topology_strategy*>(&ks.get_replication_strategy()); bool network_topology = dynamic_cast<const locator::network_topology_strategy*>(&ks.get_replication_strategy());
auto target_endpoint = get_view_natural_endpoint(ermp, network_topology, base_token, view_token); auto target_endpoint = get_view_natural_endpoint(ermp, network_topology, base_token, view_token);
auto remote_endpoints = ermp->get_pending_endpoints(view_token); auto remote_endpoints = ermp->get_pending_endpoints(view_token);

View File

@@ -30,3 +30,24 @@ async def test_tablet_mv_create(manager: ManagerClient):
await cql.run_async("CREATE TABLE test.test (pk int PRIMARY KEY, c int)") await cql.run_async("CREATE TABLE test.test (pk int PRIMARY KEY, c int)")
await cql.run_async("CREATE MATERIALIZED VIEW test.tv AS SELECT * FROM test.test WHERE c IS NOT NULL AND pk IS NOT NULL PRIMARY KEY (c, pk)") await cql.run_async("CREATE MATERIALIZED VIEW test.tv AS SELECT * FROM test.test WHERE c IS NOT NULL AND pk IS NOT NULL PRIMARY KEY (c, pk)")
await cql.run_async("DROP KEYSPACE test") await cql.run_async("DROP KEYSPACE test")
@pytest.mark.asyncio
async def test_tablet_mv_simple(manager: ManagerClient):
"""A simple test for reading and writing a materialized view on a table
stored with tablets on a one-node cluster. Because it's a one-node
cluster, we don't don't need any sophisticated mappings or pairings
to work correctly for this test to pass - everything is on this single
node anyway.
Reproduces issue #16209.
"""
servers = [await manager.server_add() for i in range(1)]
cql = manager.get_cql()
await cql.run_async("CREATE KEYSPACE test WITH replication = {'class': 'NetworkTopologyStrategy', 'replication_factor': 1, 'initial_tablets': 100}")
await cql.run_async("CREATE TABLE test.test (pk int PRIMARY KEY, c int)")
await cql.run_async("CREATE MATERIALIZED VIEW test.tv AS SELECT * FROM test.test WHERE c IS NOT NULL AND pk IS NOT NULL PRIMARY KEY (c, pk) WITH SYNCHRONOUS_UPDATES = TRUE")
await cql.run_async("INSERT INTO test.test (pk, c) VALUES (2, 3)")
# We used SYNCHRONOUS_UPDATES=TRUE, so the view should be updated:
assert [(3,2)] == list(await cql.run_async("SELECT * FROM test.tv WHERE c=3"))
await cql.run_async("DROP KEYSPACE test")