ClickHouse/tests/integration/test_cluster_copier/test_trivial.py

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

228 lines
7.2 KiB
Python
Raw Normal View History

2020-03-16 21:05:38 +00:00
import os
import sys
import time
2021-06-08 01:50:43 +00:00
import random
import string
2020-03-16 21:05:38 +00:00
2021-04-27 12:34:56 +00:00
from helpers.cluster import ClickHouseCluster
from helpers.test_tools import TSV
import kazoo
import pytest
2021-04-27 12:34:56 +00:00
import docker
2020-03-16 21:05:38 +00:00
CURRENT_TEST_DIR = os.path.dirname(os.path.abspath(__file__))
sys.path.insert(0, os.path.dirname(CURRENT_TEST_DIR))
2021-04-27 12:34:56 +00:00
2020-03-16 21:05:38 +00:00
2021-04-29 22:56:41 +00:00
COPYING_FAIL_PROBABILITY = 0.1
2020-12-23 18:59:27 +00:00
MOVING_FAIL_PROBABILITY = 0.1
2020-03-16 21:05:38 +00:00
cluster = ClickHouseCluster(__file__)
2021-06-08 01:50:43 +00:00
def generateRandomString(count):
return "".join(
random.choice(string.ascii_uppercase + string.digits) for _ in range(count)
)
2021-06-08 01:50:43 +00:00
@pytest.fixture(scope="module")
2020-03-16 21:05:38 +00:00
def started_cluster():
global cluster
try:
2021-06-08 01:50:43 +00:00
for name in ["first_trivial", "second_trivial"]:
instance = cluster.add_instance(
name,
2021-06-08 01:50:43 +00:00
main_configs=["configs/conf.d/clusters_trivial.xml"],
user_configs=["configs_two_nodes/users.xml"],
macros={
"cluster": name,
"shard": "the_only_shard",
"replica": "the_only_replica",
},
with_zookeeper=True,
)
2020-03-16 21:05:38 +00:00
cluster.start()
yield cluster
finally:
cluster.shutdown()
class TaskTrivial:
2021-06-08 01:50:43 +00:00
def __init__(self, cluster):
2020-03-16 21:05:38 +00:00
self.cluster = cluster
2021-06-08 01:50:43 +00:00
self.zk_task_path = "/clickhouse-copier/task_trivial"
self.copier_task_config = open(
os.path.join(CURRENT_TEST_DIR, "task_trivial.xml"), "r"
).read()
2020-03-16 21:05:38 +00:00
def start(self):
source = cluster.instances["first_trivial"]
destination = cluster.instances["second_trivial"]
2020-03-16 21:05:38 +00:00
for node in [source, destination]:
node.query("DROP DATABASE IF EXISTS default")
node.query("CREATE DATABASE IF NOT EXISTS default")
2020-03-16 21:05:38 +00:00
source.query(
"CREATE TABLE trivial (d UInt64, d1 UInt64 MATERIALIZED d+1)"
"ENGINE=ReplicatedMergeTree('/clickhouse/tables/source_trivial_cluster/1/trivial/{}', '1') "
"PARTITION BY d % 5 ORDER BY (d, sipHash64(d)) SAMPLE BY sipHash64(d) SETTINGS index_granularity = 16".format(
generateRandomString(10)
)
)
2020-03-16 21:05:38 +00:00
source.query(
"INSERT INTO trivial SELECT * FROM system.numbers LIMIT 1002",
Rename directory monitor concept into background INSERT (#55978) * Limit log frequence for "Skipping send data over distributed table" message After SYSTEM STOP DISTRIBUTED SENDS it will constantly print this message. Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> * Rename directory monitor concept into async INSERT Rename the following query settings (with preserving backward compatiblity, by keeping old name as an alias): - distributed_directory_monitor_sleep_time_ms -> distributed_async_insert_sleep_time_ms - distributed_directory_monitor_max_sleep_time_ms -> distributed_async_insert_max_sleep_time_ms - distributed_directory_monitor_batch -> distributed_async_insert_batch_inserts - distributed_directory_monitor_split_batch_on_failure -> distributed_async_insert_split_batch_on_failure Rename the following table settings (with preserving backward compatiblity, by keeping old name as an alias): - monitor_batch_inserts -> async_insert_batch - monitor_split_batch_on_failure -> async_insert_split_batch_on_failure - directory_monitor_sleep_time_ms -> async_insert_sleep_time_ms - directory_monitor_max_sleep_time_ms -> async_insert_max_sleep_time_ms And also update all the references: $ gg -e directory_monitor_ -e monitor_ tests docs | cut -d: -f1 | sort -u | xargs sed -e 's/distributed_directory_monitor_sleep_time_ms/distributed_async_insert_sleep_time_ms/g' -e 's/distributed_directory_monitor_max_sleep_time_ms/distributed_async_insert_max_sleep_time_ms/g' -e 's/distributed_directory_monitor_batch_inserts/distributed_async_insert_batch/g' -e 's/distributed_directory_monitor_split_batch_on_failure/distributed_async_insert_split_batch_on_failure/g' -e 's/monitor_batch_inserts/async_insert_batch/g' -e 's/monitor_split_batch_on_failure/async_insert_split_batch_on_failure/g' -e 's/monitor_sleep_time_ms/async_insert_sleep_time_ms/g' -e 's/monitor_max_sleep_time_ms/async_insert_max_sleep_time_ms/g' -i Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> * Rename async_insert for Distributed into background_insert This will avoid amigibuity between general async INSERT's and INSERT into Distributed, which are indeed background, so new term express it even better. Mostly done with: $ git di HEAD^ --name-only | xargs sed -i -e 's/distributed_async_insert/distributed_background_insert/g' -e 's/async_insert_batch/background_insert_batch/g' -e 's/async_insert_split_batch_on_failure/background_insert_split_batch_on_failure/g' -e 's/async_insert_sleep_time_ms/background_insert_sleep_time_ms/g' -e 's/async_insert_max_sleep_time_ms/background_insert_max_sleep_time_ms/g' Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> * Mark 02417_opentelemetry_insert_on_distributed_table as long CI: https://s3.amazonaws.com/clickhouse-test-reports/55978/7a6abb03a0b507e29e999cb7e04f246a119c6f28/stateless_tests_flaky_check__asan_.html Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> --------- Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com>
2023-11-01 14:09:39 +00:00
settings={"distributed_foreground_insert": 1},
)
2020-03-16 21:05:38 +00:00
def check(self):
zk = cluster.get_kazoo_client("zoo1")
2021-04-29 19:16:51 +00:00
status_data, _ = zk.get(self.zk_task_path + "/status")
assert (
status_data
== b'{"hits":{"all_partitions_count":5,"processed_partitions_count":5}}'
)
2021-04-29 19:16:51 +00:00
source = cluster.instances["first_trivial"]
destination = cluster.instances["second_trivial"]
2020-03-16 21:05:38 +00:00
assert TSV(source.query("SELECT count() FROM trivial")) == TSV("1002\n")
assert TSV(destination.query("SELECT count() FROM trivial")) == TSV("1002\n")
for node in [source, destination]:
node.query("DROP TABLE trivial")
class TaskReplicatedWithoutArguments:
def __init__(self, cluster):
self.cluster = cluster
self.zk_task_path = "/clickhouse-copier/task_trivial_without_arguments"
self.copier_task_config = open(
os.path.join(CURRENT_TEST_DIR, "task_trivial_without_arguments.xml"), "r"
).read()
def start(self):
source = cluster.instances["first_trivial"]
destination = cluster.instances["second_trivial"]
for node in [source, destination]:
node.query("DROP DATABASE IF EXISTS default")
node.query("CREATE DATABASE IF NOT EXISTS default")
source.query(
"CREATE TABLE trivial_without_arguments ON CLUSTER source_trivial_cluster (d UInt64, d1 UInt64 MATERIALIZED d+1) "
"ENGINE=ReplicatedMergeTree() "
"PARTITION BY d % 5 ORDER BY (d, sipHash64(d)) SAMPLE BY sipHash64(d) SETTINGS index_granularity = 16"
)
source.query(
"INSERT INTO trivial_without_arguments SELECT * FROM system.numbers LIMIT 1002",
Rename directory monitor concept into background INSERT (#55978) * Limit log frequence for "Skipping send data over distributed table" message After SYSTEM STOP DISTRIBUTED SENDS it will constantly print this message. Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> * Rename directory monitor concept into async INSERT Rename the following query settings (with preserving backward compatiblity, by keeping old name as an alias): - distributed_directory_monitor_sleep_time_ms -> distributed_async_insert_sleep_time_ms - distributed_directory_monitor_max_sleep_time_ms -> distributed_async_insert_max_sleep_time_ms - distributed_directory_monitor_batch -> distributed_async_insert_batch_inserts - distributed_directory_monitor_split_batch_on_failure -> distributed_async_insert_split_batch_on_failure Rename the following table settings (with preserving backward compatiblity, by keeping old name as an alias): - monitor_batch_inserts -> async_insert_batch - monitor_split_batch_on_failure -> async_insert_split_batch_on_failure - directory_monitor_sleep_time_ms -> async_insert_sleep_time_ms - directory_monitor_max_sleep_time_ms -> async_insert_max_sleep_time_ms And also update all the references: $ gg -e directory_monitor_ -e monitor_ tests docs | cut -d: -f1 | sort -u | xargs sed -e 's/distributed_directory_monitor_sleep_time_ms/distributed_async_insert_sleep_time_ms/g' -e 's/distributed_directory_monitor_max_sleep_time_ms/distributed_async_insert_max_sleep_time_ms/g' -e 's/distributed_directory_monitor_batch_inserts/distributed_async_insert_batch/g' -e 's/distributed_directory_monitor_split_batch_on_failure/distributed_async_insert_split_batch_on_failure/g' -e 's/monitor_batch_inserts/async_insert_batch/g' -e 's/monitor_split_batch_on_failure/async_insert_split_batch_on_failure/g' -e 's/monitor_sleep_time_ms/async_insert_sleep_time_ms/g' -e 's/monitor_max_sleep_time_ms/async_insert_max_sleep_time_ms/g' -i Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> * Rename async_insert for Distributed into background_insert This will avoid amigibuity between general async INSERT's and INSERT into Distributed, which are indeed background, so new term express it even better. Mostly done with: $ git di HEAD^ --name-only | xargs sed -i -e 's/distributed_async_insert/distributed_background_insert/g' -e 's/async_insert_batch/background_insert_batch/g' -e 's/async_insert_split_batch_on_failure/background_insert_split_batch_on_failure/g' -e 's/async_insert_sleep_time_ms/background_insert_sleep_time_ms/g' -e 's/async_insert_max_sleep_time_ms/background_insert_max_sleep_time_ms/g' Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> * Mark 02417_opentelemetry_insert_on_distributed_table as long CI: https://s3.amazonaws.com/clickhouse-test-reports/55978/7a6abb03a0b507e29e999cb7e04f246a119c6f28/stateless_tests_flaky_check__asan_.html Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com> --------- Signed-off-by: Azat Khuzhin <a.khuzhin@semrush.com>
2023-11-01 14:09:39 +00:00
settings={"distributed_foreground_insert": 1},
)
def check(self):
zk = cluster.get_kazoo_client("zoo1")
status_data, _ = zk.get(self.zk_task_path + "/status")
assert (
status_data
== b'{"hits":{"all_partitions_count":5,"processed_partitions_count":5}}'
)
source = cluster.instances["first_trivial"]
destination = cluster.instances["second_trivial"]
assert TSV(
source.query("SELECT count() FROM trivial_without_arguments")
) == TSV("1002\n")
assert TSV(
destination.query("SELECT count() FROM trivial_without_arguments")
) == TSV("1002\n")
for node in [source, destination]:
node.query("DROP TABLE trivial_without_arguments")
2021-06-07 10:49:01 +00:00
def execute_task(started_cluster, task, cmd_options):
2020-03-16 21:05:38 +00:00
task.start()
zk = started_cluster.get_kazoo_client("zoo1")
2020-10-02 16:54:07 +00:00
print("Use ZooKeeper server: {}:{}".format(zk.hosts[0][0], zk.hosts[0][1]))
2020-03-16 21:05:38 +00:00
2021-04-27 12:34:56 +00:00
try:
zk.delete("/clickhouse-copier", recursive=True)
except kazoo.exceptions.NoNodeError:
print("No node /clickhouse-copier. It is Ok in first test.")
2020-03-16 21:05:38 +00:00
zk_task_path = task.zk_task_path
zk.ensure_path(zk_task_path)
2021-04-27 12:34:56 +00:00
zk.create(zk_task_path + "/description", task.copier_task_config.encode())
2020-03-16 21:05:38 +00:00
# Run cluster-copier processes on each node
2021-06-07 10:49:01 +00:00
docker_api = started_cluster.docker_client.api
2020-03-16 21:05:38 +00:00
copiers_exec_ids = []
cmd = [
"/usr/bin/clickhouse",
"copier",
"--config",
"/etc/clickhouse-server/config-copier.xml",
"--task-path",
zk_task_path,
"--base-dir",
"/var/log/clickhouse-server/copier",
]
2020-03-16 21:05:38 +00:00
cmd += cmd_options
2021-06-07 10:49:01 +00:00
copiers = list(started_cluster.instances.keys())
2020-03-16 21:05:38 +00:00
2021-04-27 12:34:56 +00:00
for instance_name in copiers:
2021-06-07 10:49:01 +00:00
instance = started_cluster.instances[instance_name]
2020-03-16 21:05:38 +00:00
container = instance.get_docker_handle()
instance.copy_file_to_container(
os.path.join(CURRENT_TEST_DIR, "configs/config-copier.xml"),
"/etc/clickhouse-server/config-copier.xml",
)
2021-04-27 12:34:56 +00:00
print("Copied copier config to {}".format(instance.name))
2020-03-16 21:05:38 +00:00
exec_id = docker_api.exec_create(container.id, cmd, stderr=True)
output = docker_api.exec_start(exec_id).decode("utf8")
2021-04-27 12:34:56 +00:00
print(output)
2020-03-16 21:05:38 +00:00
copiers_exec_ids.append(exec_id)
print(
"Copier for {} ({}) has started".format(instance.name, instance.ip_address)
)
2020-03-16 21:05:38 +00:00
# Wait for copiers stopping and check their return codes
2021-04-27 12:34:56 +00:00
for exec_id, instance_name in zip(copiers_exec_ids, copiers):
2021-06-07 10:49:01 +00:00
instance = started_cluster.instances[instance_name]
2020-03-16 21:05:38 +00:00
while True:
res = docker_api.exec_inspect(exec_id)
if not res["Running"]:
2020-03-16 21:05:38 +00:00
break
2021-04-27 12:34:56 +00:00
time.sleep(0.5)
2020-03-16 21:05:38 +00:00
assert res["ExitCode"] == 0, "Instance: {} ({}). Info: {}".format(
instance.name, instance.ip_address, repr(res)
)
2020-03-16 21:05:38 +00:00
try:
task.check()
finally:
zk.delete(zk_task_path, recursive=True)
# Tests
2021-06-08 01:50:43 +00:00
def test_trivial_copy(started_cluster):
execute_task(started_cluster, TaskTrivial(started_cluster), [])
2020-04-21 17:37:40 +00:00
def test_trivial_without_arguments(started_cluster):
2021-06-07 10:49:01 +00:00
execute_task(started_cluster, TaskReplicatedWithoutArguments(started_cluster), [])