ClickHouse/tests/integration/test_modify_engine_on_restart/test.py

Ignoring revisions in .git-blame-ignore-revs. Click here to bypass and see the normal blame view.

162 lines
4.1 KiB
Python
Raw Normal View History

2023-12-12 18:45:00 +00:00
import pytest
from test_modify_engine_on_restart.common import check_flags_deleted, set_convert_flags
2023-12-12 18:45:00 +00:00
from helpers.cluster import ClickHouseCluster
cluster = ClickHouseCluster(__file__)
ch1 = cluster.add_instance(
"ch1",
main_configs=[
"configs/config.d/clusters.xml",
"configs/config.d/distributed_ddl.xml",
],
with_zookeeper=True,
macros={"replica": "node1"},
stay_alive=True,
)
ch2 = cluster.add_instance(
"ch2",
main_configs=[
"configs/config.d/clusters.xml",
"configs/config.d/distributed_ddl.xml",
],
with_zookeeper=True,
macros={"replica": "node2"},
)
database_name = "modify_engine"
2023-12-16 10:48:25 +00:00
2023-12-12 18:45:00 +00:00
@pytest.fixture(scope="module")
def started_cluster():
try:
cluster.start()
yield cluster
finally:
cluster.shutdown()
2023-12-16 10:48:25 +00:00
2023-12-12 18:45:00 +00:00
def q(node, query):
2023-12-16 10:48:25 +00:00
return node.query(database=database_name, sql=query)
2023-12-12 18:45:00 +00:00
def create_tables():
# MergeTree table that will be converted
q(
ch1,
2023-12-16 10:48:25 +00:00
"CREATE TABLE rmt ( A Int64, D Date, S String ) ENGINE MergeTree() PARTITION BY toYYYYMM(D) ORDER BY A;",
2023-12-12 18:45:00 +00:00
)
2023-12-16 10:48:25 +00:00
q(ch1, "INSERT INTO rmt SELECT number, today(), '' FROM numbers(1e6);")
q(ch1, "INSERT INTO rmt SELECT number, today()-60, '' FROM numbers(1e5);")
2023-12-12 18:45:00 +00:00
# ReplacingMergeTree table that will be converted to check unusual engine kinds
q(
ch1,
2023-12-16 10:48:25 +00:00
"CREATE TABLE replacing ( A Int64, D Date, S String ) ENGINE ReplacingMergeTree() PARTITION BY toYYYYMM(D) ORDER BY A;",
2023-12-12 18:45:00 +00:00
)
2023-12-16 10:48:25 +00:00
q(ch1, "INSERT INTO replacing SELECT number, today(), '' FROM numbers(1e6);")
q(ch1, "INSERT INTO replacing SELECT number, today()-60, '' FROM numbers(1e5);")
2023-12-12 18:45:00 +00:00
# MergeTree table that will not be converted
q(
ch1,
2023-12-16 10:48:25 +00:00
"CREATE TABLE mt ( A Int64, D Date, S String ) ENGINE MergeTree() PARTITION BY toYYYYMM(D) ORDER BY A;",
2023-12-12 18:45:00 +00:00
)
# Not MergeTree table
2023-12-16 10:48:25 +00:00
q(ch1, "CREATE TABLE log ( A Int64, D Date, S String ) ENGINE Log;")
2023-12-12 18:45:00 +00:00
2023-12-14 20:01:50 +00:00
def check_tables(converted):
engine_prefix = ""
2023-12-16 10:48:25 +00:00
if converted:
2023-12-14 20:01:50 +00:00
engine_prefix = "Replicated"
2023-12-12 18:45:00 +00:00
# Check tables exists
2023-12-16 10:48:25 +00:00
assert (
q(
ch1,
"SHOW TABLES",
).strip()
== "log\nmt\nreplacing\nrmt"
)
2023-12-12 18:45:00 +00:00
# Check engines
2023-12-16 10:48:25 +00:00
assert (
q(
ch1,
f"SELECT name, engine FROM system.tables WHERE database = '{database_name}' AND (name != 'log' AND name != 'mt')",
).strip()
== f"replacing\t{engine_prefix}ReplacingMergeTree\nrmt\t{engine_prefix}MergeTree"
)
assert (
q(
ch1,
f"SELECT name, engine FROM system.tables WHERE database = '{database_name}' AND (name = 'log' OR name = 'mt')",
).strip()
== "log\tLog\nmt\tMergeTree"
)
2023-12-12 18:45:00 +00:00
# Check values
2023-12-14 20:01:50 +00:00
for table in ["rmt", "replacing"]:
2023-12-16 10:48:25 +00:00
assert (
q(
ch1,
f"SELECT count() FROM {table}",
).strip()
== "1100000"
)
2023-12-14 20:01:50 +00:00
2023-12-12 18:45:00 +00:00
def check_replica_added():
# Add replica to check if zookeeper path is correct and consistent with table uuid
uuid = q(
ch1,
2023-12-16 10:48:25 +00:00
f"SELECT uuid FROM system.tables WHERE table = 'rmt' AND database = '{database_name}'",
2023-12-12 18:45:00 +00:00
).strip()
q(
ch2,
2023-12-16 10:48:25 +00:00
f"CREATE TABLE rmt ( A Int64, D Date, S String ) ENGINE ReplicatedMergeTree('/clickhouse/tables/{uuid}/{{shard}}', '{{replica}}') PARTITION BY toYYYYMM(D) ORDER BY A",
2023-12-12 18:45:00 +00:00
)
2023-12-16 10:48:25 +00:00
2023-12-14 20:01:50 +00:00
ch2.query(database=database_name, sql="SYSTEM SYNC REPLICA rmt", timeout=20)
2023-12-12 18:45:00 +00:00
# Check values
2023-12-16 10:48:25 +00:00
assert (
q(
ch2,
f"SELECT count() FROM rmt",
).strip()
== "1100000"
)
2023-12-12 18:45:00 +00:00
def test_modify_engine_on_restart(started_cluster):
ch1.query("CREATE DATABASE " + database_name + " ON CLUSTER cluster")
2023-12-16 10:48:25 +00:00
2023-12-12 18:45:00 +00:00
create_tables()
2023-12-14 20:01:50 +00:00
check_tables(False)
2023-12-12 18:45:00 +00:00
ch1.restart_clickhouse()
2023-12-14 20:01:50 +00:00
check_tables(False)
2023-12-12 18:45:00 +00:00
set_convert_flags(ch1, database_name, ["rmt", "replacing", "log"])
2023-12-12 18:45:00 +00:00
ch1.restart_clickhouse()
check_flags_deleted(ch1, database_name, ["rmt", "replacing"])
2023-12-14 20:01:50 +00:00
check_tables(True)
2023-12-12 18:45:00 +00:00
check_replica_added()
2023-12-14 20:01:50 +00:00
ch1.restart_clickhouse()
2023-12-16 10:48:25 +00:00
check_tables(True)