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
54 lines
2.3 KiB
Python
54 lines
2.3 KiB
Python
#
|
|
# Copyright (C) 2023-present ScyllaDB
|
|
#
|
|
# SPDX-License-Identifier: AGPL-3.0-or-later
|
|
#
|
|
|
|
# Tests for interaction of materialized views with *tablets*
|
|
|
|
from test.pylib.manager_client import ManagerClient
|
|
|
|
import pytest
|
|
import asyncio
|
|
import logging
|
|
|
|
|
|
logger = logging.getLogger(__name__)
|
|
|
|
|
|
@pytest.mark.asyncio
|
|
async def test_tablet_mv_create(manager: ManagerClient):
|
|
"""A basic test for creating a materialized view on a table stored
|
|
with tablets on a one-node cluster. We just create the view and
|
|
delete it - that's it, we don't read or write the table.
|
|
Reproduces issue #16194.
|
|
"""
|
|
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)")
|
|
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")
|