2023-12-19 14:16:23 +00:00
|
|
|
import time
|
|
|
|
import pytest
|
|
|
|
from helpers.cluster import ClickHouseCluster
|
|
|
|
|
|
|
|
cluster = ClickHouseCluster(__file__)
|
|
|
|
|
|
|
|
node_with_backoff = cluster.add_instance(
|
|
|
|
"node_with_backoff",
|
|
|
|
macros={"cluster": "test_cluster"},
|
2024-02-04 11:24:26 +00:00
|
|
|
main_configs=["configs/config.d/backoff_mutation_policy.xml"],
|
2023-12-19 14:16:23 +00:00
|
|
|
with_zookeeper=True,
|
2024-02-04 11:24:26 +00:00
|
|
|
stay_alive=True,
|
2023-12-19 14:16:23 +00:00
|
|
|
)
|
|
|
|
|
|
|
|
node_no_backoff = cluster.add_instance(
|
|
|
|
"node_no_backoff",
|
|
|
|
macros={"cluster": "test_cluster"},
|
2024-02-20 08:39:11 +00:00
|
|
|
main_configs=["configs/config.d/no_backoff_mutation_policy.xml"],
|
2023-12-19 14:16:23 +00:00
|
|
|
with_zookeeper=True,
|
|
|
|
)
|
|
|
|
|
2024-03-12 11:43:08 +00:00
|
|
|
REPLICATED_POSTPONE_MUTATION_LOG = (
|
2023-12-19 14:16:23 +00:00
|
|
|
"According to exponential backoff policy, put aside this log entry"
|
|
|
|
)
|
2024-03-12 11:43:08 +00:00
|
|
|
POSTPONE_MUTATION_LOG = (
|
2023-12-19 14:16:23 +00:00
|
|
|
"According to exponential backoff policy, do not perform mutations for the part"
|
|
|
|
)
|
|
|
|
|
|
|
|
all_nodes = [node_with_backoff, node_no_backoff]
|
|
|
|
|
|
|
|
|
|
|
|
def prepare_cluster(use_replicated_table):
|
|
|
|
for node in all_nodes:
|
|
|
|
node.query("DROP TABLE IF EXISTS test_mutations SYNC")
|
|
|
|
|
|
|
|
engine = (
|
|
|
|
"ReplicatedMergeTree('/clickhouse/{cluster}/tables/test/test_mutations', '{instance}')"
|
|
|
|
if use_replicated_table
|
|
|
|
else "MergeTree()"
|
|
|
|
)
|
|
|
|
|
|
|
|
for node in all_nodes:
|
2024-02-04 11:24:26 +00:00
|
|
|
node.rotate_logs()
|
2023-12-19 14:16:23 +00:00
|
|
|
node.query(f"CREATE TABLE test_mutations(x UInt32) ENGINE {engine} ORDER BY x")
|
|
|
|
node.query("INSERT INTO test_mutations SELECT * FROM system.numbers LIMIT 10")
|
|
|
|
|
|
|
|
|
|
|
|
@pytest.fixture(scope="module")
|
|
|
|
def started_cluster():
|
|
|
|
try:
|
|
|
|
cluster.start()
|
|
|
|
yield cluster
|
|
|
|
|
|
|
|
finally:
|
|
|
|
cluster.shutdown()
|
|
|
|
|
|
|
|
|
|
|
|
@pytest.mark.parametrize(
|
2024-03-12 11:43:08 +00:00
|
|
|
("node, found_in_log"),
|
2023-12-19 14:16:23 +00:00
|
|
|
[
|
2024-03-12 11:43:08 +00:00
|
|
|
(
|
|
|
|
node_with_backoff,
|
|
|
|
True,
|
|
|
|
),
|
|
|
|
(
|
|
|
|
node_no_backoff,
|
|
|
|
False,
|
|
|
|
),
|
2023-12-19 14:16:23 +00:00
|
|
|
],
|
|
|
|
)
|
2024-03-12 11:43:08 +00:00
|
|
|
def test_exponential_backoff_with_merge_tree(started_cluster, node, found_in_log):
|
2023-12-19 14:16:23 +00:00
|
|
|
prepare_cluster(False)
|
|
|
|
|
2024-03-12 11:43:08 +00:00
|
|
|
def check_logs():
|
|
|
|
if found_in_log:
|
|
|
|
assert node.wait_for_log_line(POSTPONE_MUTATION_LOG)
|
|
|
|
# Do not rotate the logs when we are checking the absence of a log message
|
|
|
|
node.rotate_logs()
|
|
|
|
else:
|
|
|
|
# Best effort, but when it fails, then the logs for sure contain the problematic message
|
|
|
|
assert not node.contains_in_log(POSTPONE_MUTATION_LOG)
|
|
|
|
|
2023-12-19 14:16:23 +00:00
|
|
|
# Executing incorrect mutation.
|
|
|
|
node.query(
|
2024-02-28 00:12:34 +00:00
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM notexist_table) SETTINGS allow_nondeterministic_mutations=1"
|
2023-12-19 14:16:23 +00:00
|
|
|
)
|
|
|
|
|
2024-03-12 11:43:08 +00:00
|
|
|
check_logs()
|
2023-12-19 14:16:23 +00:00
|
|
|
|
|
|
|
node.query("KILL MUTATION WHERE table='test_mutations'")
|
|
|
|
# Check that after kill new parts mutations are postponing.
|
|
|
|
node.query(
|
2024-02-28 00:12:34 +00:00
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM notexist_table) SETTINGS allow_nondeterministic_mutations=1"
|
2023-12-19 14:16:23 +00:00
|
|
|
)
|
|
|
|
|
2024-03-12 11:43:08 +00:00
|
|
|
check_logs()
|
2023-12-19 14:16:23 +00:00
|
|
|
|
|
|
|
|
|
|
|
def test_exponential_backoff_with_replicated_tree(started_cluster):
|
|
|
|
prepare_cluster(True)
|
|
|
|
|
2024-01-30 10:37:08 +00:00
|
|
|
node_with_backoff.query(
|
2024-02-28 00:12:34 +00:00
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM notexist_table) SETTINGS allow_nondeterministic_mutations=1"
|
2023-12-19 14:16:23 +00:00
|
|
|
)
|
|
|
|
|
2024-03-12 11:43:08 +00:00
|
|
|
assert node_with_backoff.wait_for_log_line(REPLICATED_POSTPONE_MUTATION_LOG)
|
|
|
|
assert not node_no_backoff.contains_in_log(REPLICATED_POSTPONE_MUTATION_LOG)
|
2023-12-19 14:16:23 +00:00
|
|
|
|
|
|
|
|
2024-03-12 11:43:08 +00:00
|
|
|
def test_exponential_backoff_create_dependent_table(started_cluster):
|
2023-12-19 14:16:23 +00:00
|
|
|
prepare_cluster(False)
|
|
|
|
|
|
|
|
# Executing incorrect mutation.
|
2024-03-12 11:43:08 +00:00
|
|
|
node_with_backoff.query(
|
2023-12-19 14:16:23 +00:00
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM dep_table) SETTINGS allow_nondeterministic_mutations=1"
|
|
|
|
)
|
2024-02-18 13:37:12 +00:00
|
|
|
|
2023-12-19 14:16:23 +00:00
|
|
|
# Creating dependent table for mutation.
|
2024-03-12 11:54:56 +00:00
|
|
|
node_with_backoff.query(
|
|
|
|
"CREATE TABLE dep_table(x UInt32) ENGINE MergeTree() ORDER BY x"
|
|
|
|
)
|
2023-12-19 14:16:23 +00:00
|
|
|
|
2024-02-18 13:37:12 +00:00
|
|
|
retry_count = 100
|
|
|
|
no_unfinished_mutation = False
|
2024-02-18 14:46:48 +00:00
|
|
|
for _ in range(0, retry_count):
|
2024-03-12 11:54:56 +00:00
|
|
|
if (
|
|
|
|
node_with_backoff.query(
|
|
|
|
"SELECT count() FROM system.mutations WHERE is_done=0"
|
|
|
|
)
|
|
|
|
== "0\n"
|
|
|
|
):
|
2024-02-18 13:37:12 +00:00
|
|
|
no_unfinished_mutation = True
|
|
|
|
break
|
|
|
|
|
|
|
|
assert no_unfinished_mutation
|
2024-03-12 11:54:56 +00:00
|
|
|
node_with_backoff.query("DROP TABLE IF EXISTS dep_table SYNC")
|
2024-02-04 11:24:26 +00:00
|
|
|
|
|
|
|
|
|
|
|
def test_exponential_backoff_setting_override(started_cluster):
|
|
|
|
node = node_with_backoff
|
|
|
|
node.rotate_logs()
|
|
|
|
node.query("DROP TABLE IF EXISTS test_mutations SYNC")
|
|
|
|
node.query(
|
2024-02-18 13:37:12 +00:00
|
|
|
"CREATE TABLE test_mutations(x UInt32) ENGINE=MergeTree() ORDER BY x SETTINGS max_postpone_time_for_failed_mutations_ms=0"
|
2024-02-04 11:24:26 +00:00
|
|
|
)
|
|
|
|
node.query("INSERT INTO test_mutations SELECT * FROM system.numbers LIMIT 10")
|
|
|
|
|
|
|
|
# Executing incorrect mutation.
|
|
|
|
node.query(
|
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM dep_table) SETTINGS allow_nondeterministic_mutations=1"
|
|
|
|
)
|
2024-03-12 11:43:08 +00:00
|
|
|
assert not node.contains_in_log(POSTPONE_MUTATION_LOG)
|
2024-01-30 10:37:08 +00:00
|
|
|
|
|
|
|
|
|
|
|
@pytest.mark.parametrize(
|
|
|
|
("replicated_table"),
|
|
|
|
[
|
|
|
|
(False),
|
|
|
|
(True),
|
|
|
|
],
|
|
|
|
)
|
2024-02-04 11:24:26 +00:00
|
|
|
def test_backoff_clickhouse_restart(started_cluster, replicated_table):
|
2024-01-30 10:37:08 +00:00
|
|
|
prepare_cluster(replicated_table)
|
2024-02-04 11:24:26 +00:00
|
|
|
node = node_with_backoff
|
2024-01-30 10:37:08 +00:00
|
|
|
|
|
|
|
# Executing incorrect mutation.
|
|
|
|
node.query(
|
2024-02-04 11:24:26 +00:00
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM dep_table) SETTINGS allow_nondeterministic_mutations=1"
|
2024-01-30 10:37:08 +00:00
|
|
|
)
|
2024-02-04 11:24:26 +00:00
|
|
|
assert node.wait_for_log_line(
|
2024-03-12 11:43:08 +00:00
|
|
|
REPLICATED_POSTPONE_MUTATION_LOG if replicated_table else POSTPONE_MUTATION_LOG
|
2024-01-30 10:37:08 +00:00
|
|
|
)
|
2024-02-04 11:24:26 +00:00
|
|
|
|
2024-01-30 10:37:08 +00:00
|
|
|
node.restart_clickhouse()
|
|
|
|
node.rotate_logs()
|
2024-02-04 11:24:26 +00:00
|
|
|
|
|
|
|
assert node.wait_for_log_line(
|
2024-03-12 11:43:08 +00:00
|
|
|
REPLICATED_POSTPONE_MUTATION_LOG if replicated_table else POSTPONE_MUTATION_LOG
|
2024-01-30 10:37:08 +00:00
|
|
|
)
|
2024-02-18 13:37:12 +00:00
|
|
|
|
|
|
|
|
|
|
|
@pytest.mark.parametrize(
|
|
|
|
("replicated_table"),
|
|
|
|
[
|
|
|
|
(False),
|
|
|
|
(True),
|
|
|
|
],
|
|
|
|
)
|
|
|
|
def test_no_backoff_after_killing_mutation(started_cluster, replicated_table):
|
|
|
|
prepare_cluster(replicated_table)
|
|
|
|
node = node_with_backoff
|
|
|
|
|
|
|
|
# Executing incorrect mutation.
|
|
|
|
node.query(
|
|
|
|
"ALTER TABLE test_mutations DELETE WHERE x IN (SELECT x FROM dep_table) SETTINGS allow_nondeterministic_mutations=1"
|
|
|
|
)
|
|
|
|
# Executing correct mutation.
|
2024-02-18 14:46:48 +00:00
|
|
|
node.query("ALTER TABLE test_mutations DELETE WHERE x=1")
|
2024-02-18 13:37:12 +00:00
|
|
|
assert node.wait_for_log_line(
|
2024-03-12 11:43:08 +00:00
|
|
|
REPLICATED_POSTPONE_MUTATION_LOG if replicated_table else POSTPONE_MUTATION_LOG
|
2024-02-18 13:37:12 +00:00
|
|
|
)
|
2024-02-18 14:46:48 +00:00
|
|
|
mutation_ids = node.query("select mutation_id from system.mutations").split()
|
2024-02-18 13:37:12 +00:00
|
|
|
|
|
|
|
node.query(
|
|
|
|
f"KILL MUTATION WHERE table = 'test_mutations' AND mutation_id = '{mutation_ids[0]}'"
|
|
|
|
)
|
|
|
|
node.rotate_logs()
|
2024-02-18 14:46:48 +00:00
|
|
|
assert not node.contains_in_log(
|
2024-03-12 11:43:08 +00:00
|
|
|
REPLICATED_POSTPONE_MUTATION_LOG if replicated_table else POSTPONE_MUTATION_LOG
|
2024-02-18 14:46:48 +00:00
|
|
|
)
|