ClickHouse/tests/integration/test_cluster_copier/test_three_nodes.py

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

287 lines
9.6 KiB
Python
Raw Normal View History

2021-04-22 23:13:05 +00:00
import os
import sys
import time
import logging
import pytest
from helpers.cluster import ClickHouseCluster
from helpers.test_tools import TSV
import docker
CURRENT_TEST_DIR = os.path.dirname(os.path.abspath(__file__))
sys.path.insert(0, os.path.dirname(CURRENT_TEST_DIR))
cluster = ClickHouseCluster(__file__)
2021-04-22 23:13:05 +00:00
@pytest.fixture(scope="module")
def started_cluster():
global cluster
try:
for name in ["first", "second", "third"]:
cluster.add_instance(
name,
main_configs=[
"configs_three_nodes/conf.d/clusters.xml",
"configs_three_nodes/conf.d/ddl.xml",
],
user_configs=["configs_three_nodes/users.xml"],
with_zookeeper=True,
)
2021-04-22 23:13:05 +00:00
cluster.start()
yield cluster
finally:
cluster.shutdown()
2021-04-22 23:13:05 +00:00
class Task:
def __init__(self, cluster):
self.cluster = cluster
self.zk_task_path = "/clickhouse-copier/task"
2021-04-22 23:13:05 +00:00
self.container_task_file = "/task_taxi_data.xml"
for instance_name, _ in cluster.instances.items():
instance = cluster.instances[instance_name]
instance.copy_file_to_container(
os.path.join(CURRENT_TEST_DIR, "./task_taxi_data.xml"),
self.container_task_file,
)
logging.debug(
f"Copied task file to container of '{instance_name}' instance. Path {self.container_task_file}"
)
2021-04-22 23:13:05 +00:00
def start(self):
2021-06-08 01:50:43 +00:00
for name in ["first", "second", "third"]:
node = cluster.instances[name]
node.query("DROP DATABASE IF EXISTS dailyhistory SYNC;")
node.query("DROP DATABASE IF EXISTS monthlyhistory SYNC;")
first = cluster.instances["first"]
2021-04-22 23:13:05 +00:00
# daily partition database
2021-06-16 12:31:19 +00:00
first.query("CREATE DATABASE IF NOT EXISTS dailyhistory on cluster events;")
first.query(
"""CREATE TABLE dailyhistory.yellow_tripdata_staging ON CLUSTER events
2021-04-22 23:13:05 +00:00
(
id UUID DEFAULT generateUUIDv4(),
vendor_id String,
tpep_pickup_datetime DateTime('UTC'),
2021-06-01 22:03:08 +00:00
tpep_dropoff_datetime DateTime('UTC'),
2021-04-22 23:13:05 +00:00
passenger_count Nullable(Float64),
trip_distance String,
pickup_longitude Float64,
pickup_latitude Float64,
rate_code_id String,
store_and_fwd_flag String,
dropoff_longitude Float64,
dropoff_latitude Float64,
payment_type String,
fare_amount String,
extra String,
mta_tax String,
tip_amount String,
tolls_amount String,
improvement_surcharge String,
total_amount String,
pickup_location_id String,
dropoff_location_id String,
congestion_surcharge String,
junk1 String, junk2 String
)
Engine = ReplacingMergeTree()
PRIMARY KEY (tpep_pickup_datetime, id)
ORDER BY (tpep_pickup_datetime, id)
PARTITION BY (toYYYYMMDD(tpep_pickup_datetime))"""
)
2021-04-22 23:13:05 +00:00
first.query(
"""CREATE TABLE dailyhistory.yellow_tripdata
2021-04-22 23:13:05 +00:00
ON CLUSTER events
AS dailyhistory.yellow_tripdata_staging
ENGINE = Distributed('events', 'dailyhistory', yellow_tripdata_staging, sipHash64(id) % 3);"""
)
2021-04-22 23:13:05 +00:00
first.query(
"""INSERT INTO dailyhistory.yellow_tripdata
2021-04-22 23:13:05 +00:00
SELECT * FROM generateRandom(
'id UUID DEFAULT generateUUIDv4(),
vendor_id String,
tpep_pickup_datetime DateTime(\\'UTC\\'),
2021-06-01 22:03:08 +00:00
tpep_dropoff_datetime DateTime(\\'UTC\\'),
2021-04-22 23:13:05 +00:00
passenger_count Nullable(Float64),
trip_distance String,
pickup_longitude Float64,
pickup_latitude Float64,
rate_code_id String,
store_and_fwd_flag String,
dropoff_longitude Float64,
dropoff_latitude Float64,
payment_type String,
fare_amount String,
extra String,
mta_tax String,
tip_amount String,
tolls_amount String,
improvement_surcharge String,
total_amount String,
pickup_location_id String,
dropoff_location_id String,
congestion_surcharge String,
junk1 String,
junk2 String',
1, 10, 2) LIMIT 50;"""
)
2021-04-22 23:13:05 +00:00
# monthly partition database
2021-06-16 12:31:19 +00:00
first.query("create database IF NOT EXISTS monthlyhistory on cluster events;")
first.query(
"""CREATE TABLE monthlyhistory.yellow_tripdata_staging ON CLUSTER events
2021-04-22 23:13:05 +00:00
(
id UUID DEFAULT generateUUIDv4(),
vendor_id String,
tpep_pickup_datetime DateTime('UTC'),
tpep_dropoff_datetime DateTime('UTC'),
passenger_count Nullable(Float64),
trip_distance String,
pickup_longitude Float64,
pickup_latitude Float64,
rate_code_id String,
store_and_fwd_flag String,
dropoff_longitude Float64,
dropoff_latitude Float64,
payment_type String,
fare_amount String,
extra String,
mta_tax String,
tip_amount String,
tolls_amount String,
improvement_surcharge String,
total_amount String,
pickup_location_id String,
dropoff_location_id String,
congestion_surcharge String,
junk1 String,
junk2 String
2021-06-01 22:03:08 +00:00
)
2021-04-22 23:13:05 +00:00
Engine = ReplacingMergeTree()
PRIMARY KEY (tpep_pickup_datetime, id)
ORDER BY (tpep_pickup_datetime, id)
PARTITION BY (pickup_location_id, toYYYYMM(tpep_pickup_datetime))"""
)
2021-04-22 23:13:05 +00:00
first.query(
"""CREATE TABLE monthlyhistory.yellow_tripdata
2021-04-22 23:13:05 +00:00
ON CLUSTER events
2021-06-01 22:03:08 +00:00
AS monthlyhistory.yellow_tripdata_staging
ENGINE = Distributed('events', 'monthlyhistory', yellow_tripdata_staging, sipHash64(id) % 3);"""
)
2021-04-22 23:13:05 +00:00
def check(self):
2021-06-16 12:31:19 +00:00
first = cluster.instances["first"]
a = TSV(first.query("SELECT count() from dailyhistory.yellow_tripdata"))
b = TSV(first.query("SELECT count() from monthlyhistory.yellow_tripdata"))
2021-04-22 23:13:05 +00:00
assert a == b, "Distributed tables"
for instance_name, instance in cluster.instances.items():
instance = cluster.instances[instance_name]
a = instance.query(
"SELECT count() from dailyhistory.yellow_tripdata_staging"
)
b = instance.query(
"SELECT count() from monthlyhistory.yellow_tripdata_staging"
)
2021-04-22 23:13:05 +00:00
assert a == b, "MergeTree tables on each shard"
a = TSV(
instance.query(
"SELECT sipHash64(*) from dailyhistory.yellow_tripdata_staging ORDER BY id"
)
)
b = TSV(
instance.query(
"SELECT sipHash64(*) from monthlyhistory.yellow_tripdata_staging ORDER BY id"
)
)
2021-04-22 23:13:05 +00:00
assert a == b, "Data on each shard"
2021-06-08 01:50:43 +00:00
for name in ["first", "second", "third"]:
node = cluster.instances[name]
node.query("DROP DATABASE IF EXISTS dailyhistory SYNC;")
node.query("DROP DATABASE IF EXISTS monthlyhistory SYNC;")
2021-06-07 10:49:01 +00:00
def execute_task(started_cluster, task, cmd_options):
2021-04-22 23:13:05 +00:00
task.start()
zk = started_cluster.get_kazoo_client("zoo1")
2021-06-11 12:00:40 +00:00
logging.debug("Use ZooKeeper server: {}:{}".format(zk.hosts[0][0], zk.hosts[0][1]))
2021-04-22 23:13:05 +00:00
# Run cluster-copier processes on each node
2021-06-11 12:00:40 +00:00
docker_api = started_cluster.docker_client.api
2021-04-22 23:13:05 +00:00
copiers_exec_ids = []
cmd = [
"/usr/bin/clickhouse",
"copier",
"--config",
"/etc/clickhouse-server/config-copier.xml",
"--task-path",
task.zk_task_path,
"--task-file",
task.container_task_file,
"--task-upload-force",
"true",
"--base-dir",
"/var/log/clickhouse-server/copier",
]
2021-04-22 23:13:05 +00:00
cmd += cmd_options
2021-06-11 12:00:40 +00:00
logging.debug(f"execute_task cmd: {cmd}")
2021-04-22 23:13:05 +00:00
2021-06-11 12:00:40 +00:00
for instance_name in started_cluster.instances.keys():
2021-06-07 10:49:01 +00:00
instance = started_cluster.instances[instance_name]
2021-04-22 23:13:05 +00:00
container = instance.get_docker_handle()
instance.copy_file_to_container(
os.path.join(CURRENT_TEST_DIR, "configs_three_nodes/config-copier.xml"),
"/etc/clickhouse-server/config-copier.xml",
)
2021-04-22 23:13:05 +00:00
logging.info("Copied copier config to {}".format(instance.name))
exec_id = docker_api.exec_create(container.id, cmd, stderr=True)
output = docker_api.exec_start(exec_id).decode("utf8")
2021-04-22 23:13:05 +00:00
logging.info(output)
copiers_exec_ids.append(exec_id)
logging.info(
"Copier for {} ({}) has started".format(instance.name, instance.ip_address)
)
2021-04-22 23:13:05 +00:00
# time.sleep(1000)
# Wait for copiers stopping and check their return codes
for exec_id, instance in zip(
copiers_exec_ids, iter(started_cluster.instances.values())
):
2021-04-22 23:13:05 +00:00
while True:
res = docker_api.exec_inspect(exec_id)
if not res["Running"]:
2021-04-22 23:13:05 +00:00
break
time.sleep(1)
assert res["ExitCode"] == 0, "Instance: {} ({}). Info: {}".format(
instance.name, instance.ip_address, repr(res)
)
2021-04-22 23:13:05 +00:00
try:
task.check()
finally:
zk.delete(task.zk_task_path, recursive=True)
# Tests
@pytest.mark.timeout(600)
def test(started_cluster):
2021-06-07 10:49:01 +00:00
execute_task(started_cluster, Task(started_cluster), [])