mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-24 08:32:02 +00:00
Merge branch 'master' into enable_atomic_database_by_default
This commit is contained in:
commit
fb826557b7
1
.gitmodules
vendored
1
.gitmodules
vendored
@ -186,4 +186,3 @@
|
||||
[submodule "contrib/cyrus-sasl"]
|
||||
path = contrib/cyrus-sasl
|
||||
url = https://github.com/cyrusimap/cyrus-sasl
|
||||
branch = cyrus-sasl-2.1
|
||||
|
@ -17,5 +17,5 @@ ClickHouse is an open-source column-oriented database management system that all
|
||||
|
||||
## Upcoming Events
|
||||
|
||||
* [eBay migrating from Druid](https://us02web.zoom.us/webinar/register/tZMkfu6rpjItHtaQ1DXcgPWcSOnmM73HLGKL) on September 23, 2020.
|
||||
* [ClickHouse for Edge Analytics](https://ones2020.sched.com/event/bWPs) on September 29, 2020.
|
||||
* [ClickHouse online meetup (in Russian)](https://clck.ru/R2zB9) on October 1, 2020.
|
||||
|
@ -14,10 +14,10 @@ if (NOT ENABLE_RDKAFKA)
|
||||
return()
|
||||
endif()
|
||||
|
||||
if (NOT ARCH_ARM)
|
||||
if (NOT ARCH_ARM AND USE_LIBGSASL)
|
||||
option (USE_INTERNAL_RDKAFKA_LIBRARY "Set to FALSE to use system librdkafka instead of the bundled" ${NOT_UNBUNDLED})
|
||||
elseif(USE_INTERNAL_RDKAFKA_LIBRARY)
|
||||
message (${RECONFIGURE_MESSAGE_LEVEL} "Can't use internal librdkafka with ARCH_ARM=${ARCH_ARM}")
|
||||
message (${RECONFIGURE_MESSAGE_LEVEL} "Can't use internal librdkafka with ARCH_ARM=${ARCH_ARM} AND USE_LIBGSASL=${USE_LIBGSASL}")
|
||||
endif ()
|
||||
|
||||
if (NOT EXISTS "${ClickHouse_SOURCE_DIR}/contrib/cppkafka/CMakeLists.txt")
|
||||
|
2
contrib/cyrus-sasl
vendored
2
contrib/cyrus-sasl
vendored
@ -1 +1 @@
|
||||
Subproject commit 9995bf9d8e14f58934d9313ac64f13780d6dd3c9
|
||||
Subproject commit 6054630889fd1cd8d0659573d69badcee1e23a00
|
2
contrib/protobuf
vendored
2
contrib/protobuf
vendored
@ -1 +1 @@
|
||||
Subproject commit d6a10dd3db55d8f7f9e464db9151874cde1f79ec
|
||||
Subproject commit 445d1ae73a450b1e94622e7040989aa2048402e3
|
@ -11,3 +11,7 @@ else ()
|
||||
endif ()
|
||||
|
||||
add_subdirectory("${protobuf_SOURCE_DIR}/cmake" "${protobuf_BINARY_DIR}")
|
||||
|
||||
# We don't want to stop compilation on warnings in protobuf's headers.
|
||||
# The following line overrides the value assigned by the command target_include_directories() in libprotobuf.cmake
|
||||
set_property(TARGET libprotobuf PROPERTY INTERFACE_SYSTEM_INCLUDE_DIRECTORIES ${protobuf_SOURCE_DIR}/src)
|
||||
|
4
debian/rules
vendored
4
debian/rules
vendored
@ -36,8 +36,8 @@ endif
|
||||
|
||||
CMAKE_FLAGS += -DENABLE_UTILS=0
|
||||
|
||||
DEB_CC ?= $(shell which gcc-9 gcc-8 gcc | head -n1)
|
||||
DEB_CXX ?= $(shell which g++-9 g++-8 g++ | head -n1)
|
||||
DEB_CC ?= $(shell which gcc-10 gcc-9 gcc | head -n1)
|
||||
DEB_CXX ?= $(shell which g++-10 g++-9 g++ | head -n1)
|
||||
|
||||
ifdef DEB_CXX
|
||||
DEB_BUILD_GNU_TYPE := $(shell dpkg-architecture -qDEB_BUILD_GNU_TYPE)
|
||||
|
@ -133,10 +133,6 @@
|
||||
"name": "yandex/clickhouse-postgresql-java-client",
|
||||
"dependent": []
|
||||
},
|
||||
"docker/test/integration/kerberos_kdc": {
|
||||
"name": "yandex/clickhouse-kerberos-kdc",
|
||||
"dependent": []
|
||||
},
|
||||
"docker/test/base": {
|
||||
"name": "yandex/clickhouse-test-base",
|
||||
"dependent": [
|
||||
|
@ -97,7 +97,7 @@ ccache --zero-stats ||:
|
||||
mkdir build
|
||||
cd build
|
||||
cmake .. -DCMAKE_INSTALL_PREFIX=/usr -DCMAKE_CXX_COMPILER=clang++-10 -DCMAKE_C_COMPILER=clang-10 "${CMAKE_LIBS_CONFIG[@]}" "${FASTTEST_CMAKE_FLAGS[@]}" | ts '%Y-%m-%d %H:%M:%S' | tee /test_output/cmake_log.txt
|
||||
ninja clickhouse-bundle | ts '%Y-%m-%d %H:%M:%S' | tee /test_output/build_log.txt
|
||||
time ninja clickhouse-bundle | ts '%Y-%m-%d %H:%M:%S' | tee /test_output/build_log.txt
|
||||
ninja install | ts '%Y-%m-%d %H:%M:%S' | tee /test_output/install_log.txt
|
||||
|
||||
|
||||
@ -111,36 +111,10 @@ ln -s /test_output /var/log/clickhouse-server
|
||||
cp "$CLICKHOUSE_DIR/programs/server/config.xml" /etc/clickhouse-server/
|
||||
cp "$CLICKHOUSE_DIR/programs/server/users.xml" /etc/clickhouse-server/
|
||||
|
||||
mkdir -p /etc/clickhouse-server/dict_examples
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/zookeeper.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/listen.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/metric_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/custom_settings_prefixes.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/readonly.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/access_management.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/executable_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/macros.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/disks.xml /etc/clickhouse-server/config.d/
|
||||
#ln -s /usr/share/clickhouse-test/config/secure_ports.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/clusters.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/graphite.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/server.key /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/server.crt /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/dhparam.pem /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_configd.xml /etc/clickhouse-server/config.d/
|
||||
ln -sf /usr/share/clickhouse-test/config/client_config.xml /etc/clickhouse-client/config.xml
|
||||
|
||||
# Keep original query_masking_rules.xml
|
||||
ln -s --backup=simple --suffix=_original.xml /usr/share/clickhouse-test/config/query_masking_rules.xml /etc/clickhouse-server/config.d/
|
||||
# install tests config
|
||||
$CLICKHOUSE_DIR/tests/config/install.sh
|
||||
# doesn't support SSL
|
||||
rm -f /etc/clickhouse-server/config.d/secure_ports.xml
|
||||
|
||||
# Kill the server in case we are running locally and not in docker
|
||||
kill_clickhouse
|
||||
@ -217,7 +191,7 @@ TESTS_TO_SKIP=(
|
||||
01460_DistributedFilesToInsert
|
||||
)
|
||||
|
||||
clickhouse-test -j 4 --no-long --testname --shard --zookeeper --skip "${TESTS_TO_SKIP[@]}" 2>&1 | ts '%Y-%m-%d %H:%M:%S' | tee /test_output/test_log.txt
|
||||
time clickhouse-test -j 8 --no-long --testname --shard --zookeeper --skip "${TESTS_TO_SKIP[@]}" 2>&1 | ts '%Y-%m-%d %H:%M:%S' | tee /test_output/test_log.txt
|
||||
|
||||
|
||||
# substr is to remove semicolon after test name
|
||||
@ -235,7 +209,7 @@ then
|
||||
kill_clickhouse
|
||||
|
||||
# Clean the data so that there is no interference from the previous test run.
|
||||
rm -rvf /var/lib/clickhouse ||:
|
||||
rm -rf /var/lib/clickhouse ||:
|
||||
mkdir /var/lib/clickhouse
|
||||
|
||||
clickhouse-server --config /etc/clickhouse-server/config.xml --daemon
|
||||
|
@ -48,7 +48,7 @@ function configure
|
||||
cp -av "$repo_dir"/programs/server/config* db
|
||||
cp -av "$repo_dir"/programs/server/user* db
|
||||
# TODO figure out which ones are needed
|
||||
cp -av "$repo_dir"/tests/config/listen.xml db/config.d
|
||||
cp -av "$repo_dir"/tests/config/config.d/listen.xml db/config.d
|
||||
cp -av "$script_dir"/query-fuzzer-tweaks-users.xml db/users.d
|
||||
}
|
||||
|
||||
|
@ -16,8 +16,7 @@ RUN apt-get update \
|
||||
odbc-postgresql \
|
||||
sqlite3 \
|
||||
curl \
|
||||
tar \
|
||||
krb5-user
|
||||
tar
|
||||
RUN rm -rf \
|
||||
/var/lib/apt/lists/* \
|
||||
/var/cache/debconf \
|
||||
|
@ -1,15 +0,0 @@
|
||||
# docker build -t yandex/clickhouse-kerberos-kdc .
|
||||
|
||||
FROM centos:6.6
|
||||
# old OS to make is faster and smaller
|
||||
|
||||
RUN yum install -y krb5-server krb5-libs krb5-auth-dialog krb5-workstation
|
||||
|
||||
EXPOSE 88 749
|
||||
|
||||
RUN touch /config.sh
|
||||
# should be overwritten e.g. via docker_compose volumes
|
||||
# volumes: /some_path/my_kerberos_config.sh:/config.sh:ro
|
||||
|
||||
|
||||
ENTRYPOINT ["/bin/bash", "/config.sh"]
|
@ -1,59 +0,0 @@
|
||||
version: '2.3'
|
||||
|
||||
services:
|
||||
kafka_kerberized_zookeeper:
|
||||
image: confluentinc/cp-zookeeper:5.2.0
|
||||
# restart: always
|
||||
hostname: kafka_kerberized_zookeeper
|
||||
environment:
|
||||
ZOOKEEPER_SERVER_ID: 1
|
||||
ZOOKEEPER_CLIENT_PORT: 2181
|
||||
ZOOKEEPER_SERVERS: "kafka_kerberized_zookeeper:2888:3888"
|
||||
KAFKA_OPTS: "-Djava.security.auth.login.config=/etc/kafka/secrets/zookeeper_jaas.conf -Djava.security.krb5.conf=/etc/kafka/secrets/krb.conf -Dzookeeper.authProvider.1=org.apache.zookeeper.server.auth.SASLAuthenticationProvider -Dsun.security.krb5.debug=true"
|
||||
volumes:
|
||||
- ${KERBERIZED_KAFKA_DIR}/secrets:/etc/kafka/secrets
|
||||
- /dev/urandom:/dev/random
|
||||
depends_on:
|
||||
- kafka_kerberos
|
||||
security_opt:
|
||||
- label:disable
|
||||
|
||||
kerberized_kafka1:
|
||||
image: confluentinc/cp-kafka:5.2.0
|
||||
# restart: always
|
||||
hostname: kerberized_kafka1
|
||||
ports:
|
||||
- "9092:9092"
|
||||
- "9093:9093"
|
||||
environment:
|
||||
KAFKA_LISTENERS: OUTSIDE://:19092,UNSECURED_OUTSIDE://:19093,UNSECURED_INSIDE://:9093
|
||||
KAFKA_ADVERTISED_LISTENERS: OUTSIDE://kerberized_kafka1:19092,UNSECURED_OUTSIDE://kerberized_kafka1:19093,UNSECURED_INSIDE://localhost:9093
|
||||
# KAFKA_LISTENERS: INSIDE://kerberized_kafka1:9092,OUTSIDE://kerberized_kafka1:19092
|
||||
# KAFKA_ADVERTISED_LISTENERS: INSIDE://localhost:9092,OUTSIDE://kerberized_kafka1:19092
|
||||
KAFKA_SASL_MECHANISM_INTER_BROKER_PROTOCOL: GSSAPI
|
||||
KAFKA_SASL_ENABLED_MECHANISMS: GSSAPI
|
||||
KAFKA_SASL_KERBEROS_SERVICE_NAME: kafka
|
||||
KAFKA_LISTENER_SECURITY_PROTOCOL_MAP: OUTSIDE:SASL_PLAINTEXT,UNSECURED_OUTSIDE:PLAINTEXT,UNSECURED_INSIDE:PLAINTEXT,
|
||||
KAFKA_INTER_BROKER_LISTENER_NAME: OUTSIDE
|
||||
KAFKA_BROKER_ID: 1
|
||||
KAFKA_ZOOKEEPER_CONNECT: "kafka_kerberized_zookeeper:2181"
|
||||
KAFKA_LOG4J_LOGGERS: "kafka.controller=INFO,kafka.producer.async.DefaultEventHandler=INFO,state.change.logger=INFO"
|
||||
KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
|
||||
KAFKA_OPTS: "-Djava.security.auth.login.config=/etc/kafka/secrets/broker_jaas.conf -Djava.security.krb5.conf=/etc/kafka/secrets/krb.conf -Dsun.security.krb5.debug=true"
|
||||
volumes:
|
||||
- ${KERBERIZED_KAFKA_DIR}/secrets:/etc/kafka/secrets
|
||||
- /dev/urandom:/dev/random
|
||||
depends_on:
|
||||
- kafka_kerberized_zookeeper
|
||||
- kafka_kerberos
|
||||
security_opt:
|
||||
- label:disable
|
||||
|
||||
kafka_kerberos:
|
||||
image: yandex/clickhouse-kerberos-kdc:${DOCKER_KERBEROS_KDC_TAG}
|
||||
hostname: kafka_kerberos
|
||||
volumes:
|
||||
- ${KERBERIZED_KAFKA_DIR}/secrets:/tmp/keytab
|
||||
- ${KERBERIZED_KAFKA_DIR}/../../kerberos_image_config.sh:/config.sh
|
||||
- /dev/urandom:/dev/random
|
||||
ports: [88, 749]
|
@ -27,7 +27,6 @@ export DOCKER_MYSQL_JAVA_CLIENT_TAG=${DOCKER_MYSQL_JAVA_CLIENT_TAG:=latest}
|
||||
export DOCKER_MYSQL_JS_CLIENT_TAG=${DOCKER_MYSQL_JS_CLIENT_TAG:=latest}
|
||||
export DOCKER_MYSQL_PHP_CLIENT_TAG=${DOCKER_MYSQL_PHP_CLIENT_TAG:=latest}
|
||||
export DOCKER_POSTGRESQL_JAVA_CLIENT_TAG=${DOCKER_POSTGRESQL_JAVA_CLIENT_TAG:=latest}
|
||||
export DOCKER_KERBEROS_KDC_TAG=${DOCKER_KERBEROS_KDC_TAG:=latest}
|
||||
|
||||
cd /ClickHouse/tests/integration
|
||||
exec "$@"
|
||||
|
@ -726,8 +726,8 @@ create view shortness
|
||||
create table inconsistent_short_marking_report
|
||||
engine File(TSV, 'report/unexpected-query-duration.tsv')
|
||||
as select
|
||||
multiIf(marked_short and time > 0.1, '"short" queries must run faster than 0.02 s',
|
||||
not marked_short and time < 0.02, '"normal" queries must run longer than 0.1 s',
|
||||
multiIf(marked_short and time > 0.1, '\"short\" queries must run faster than 0.02 s',
|
||||
not marked_short and time < 0.02, '\"normal\" queries must run longer than 0.1 s',
|
||||
'') problem,
|
||||
marked_short, time,
|
||||
test, query_index, query_display_name
|
||||
@ -1065,7 +1065,7 @@ case "$stage" in
|
||||
# to collect the logs. Prefer not to restart, because addresses might change
|
||||
# and we won't be able to process trace_log data. Start in a subshell, so that
|
||||
# it doesn't interfere with the watchdog through `wait`.
|
||||
( get_profiles || restart && get_profiles ) ||:
|
||||
( get_profiles || { restart && get_profiles ; } ) ||:
|
||||
|
||||
# Kill the whole process group, because somehow when the subshell is killed,
|
||||
# the sleep inside remains alive and orphaned.
|
||||
|
@ -15,6 +15,7 @@ import sys
|
||||
import time
|
||||
import traceback
|
||||
import xml.etree.ElementTree as et
|
||||
from threading import Thread
|
||||
from scipy import stats
|
||||
|
||||
def tsv_escape(s):
|
||||
@ -23,10 +24,11 @@ def tsv_escape(s):
|
||||
parser = argparse.ArgumentParser(description='Run performance test.')
|
||||
# Explicitly decode files as UTF-8 because sometimes we have Russian characters in queries, and LANG=C is set.
|
||||
parser.add_argument('file', metavar='FILE', type=argparse.FileType('r', encoding='utf-8'), nargs=1, help='test description file')
|
||||
parser.add_argument('--host', nargs='*', default=['localhost'], help="Server hostname(s). Corresponds to '--port' options.")
|
||||
parser.add_argument('--port', nargs='*', default=[9000], help="Server port(s). Corresponds to '--host' options.")
|
||||
parser.add_argument('--host', nargs='*', default=['localhost'], help="Space-separated list of server hostname(s). Corresponds to '--port' options.")
|
||||
parser.add_argument('--port', nargs='*', default=[9000], help="Space-separated list of server port(s). Corresponds to '--host' options.")
|
||||
parser.add_argument('--runs', type=int, default=1, help='Number of query runs per server.')
|
||||
parser.add_argument('--max-queries', type=int, default=None, help='Test no more than this number of queries, chosen at random.')
|
||||
parser.add_argument('--queries-to-run', nargs='*', type=int, default=None, help='Space-separated list of indexes of queries to test.')
|
||||
parser.add_argument('--long', action='store_true', help='Do not skip the tests tagged as long.')
|
||||
parser.add_argument('--print-queries', action='store_true', help='Print test queries and exit.')
|
||||
parser.add_argument('--print-settings', action='store_true', help='Print test settings and exit.')
|
||||
@ -157,8 +159,11 @@ for t in tables:
|
||||
print(f'skipped\t{tsv_escape(skipped_message)}')
|
||||
sys.exit(0)
|
||||
|
||||
# Run create queries
|
||||
create_query_templates = [q.text for q in root.findall('create_query')]
|
||||
# Run create and fill queries. We will run them simultaneously for both servers,
|
||||
# to save time.
|
||||
# The weird search is to keep the relative order of elements, which matters, and
|
||||
# etree doesn't support the appropriate xpath query.
|
||||
create_query_templates = [q.text for q in root.findall('./*') if q.tag in ('create_query', 'fill_query')]
|
||||
create_queries = substitute_parameters(create_query_templates)
|
||||
|
||||
# Disallow temporary tables, because the clickhouse_driver reconnects on errors,
|
||||
@ -170,23 +175,34 @@ for q in create_queries:
|
||||
file = sys.stderr)
|
||||
sys.exit(1)
|
||||
|
||||
for conn_index, c in enumerate(all_connections):
|
||||
for q in create_queries:
|
||||
c.execute(q)
|
||||
print(f'create\t{conn_index}\t{c.last_query.elapsed}\t{tsv_escape(q)}')
|
||||
def do_create(connection, index, queries):
|
||||
for q in queries:
|
||||
connection.execute(q)
|
||||
print(f'create\t{index}\t{connection.last_query.elapsed}\t{tsv_escape(q)}')
|
||||
|
||||
# Run fill queries
|
||||
fill_query_templates = [q.text for q in root.findall('fill_query')]
|
||||
fill_queries = substitute_parameters(fill_query_templates)
|
||||
for conn_index, c in enumerate(all_connections):
|
||||
for q in fill_queries:
|
||||
c.execute(q)
|
||||
print(f'fill\t{conn_index}\t{c.last_query.elapsed}\t{tsv_escape(q)}')
|
||||
threads = [Thread(target = do_create, args = (connection, index, create_queries))
|
||||
for index, connection in enumerate(all_connections)]
|
||||
|
||||
# Run the queries in randomized order, but preserve their indexes as specified
|
||||
# in the test XML. To avoid using too much time, limit the number of queries
|
||||
# we run per test.
|
||||
queries_to_run = random.sample(range(0, len(test_queries)), min(len(test_queries), args.max_queries or len(test_queries)))
|
||||
for t in threads:
|
||||
t.start()
|
||||
|
||||
for t in threads:
|
||||
t.join()
|
||||
|
||||
queries_to_run = range(0, len(test_queries))
|
||||
|
||||
if args.max_queries:
|
||||
# If specified, test a limited number of queries chosen at random.
|
||||
queries_to_run = random.sample(range(0, len(test_queries)), min(len(test_queries), args.max_queries))
|
||||
|
||||
if args.queries_to_run:
|
||||
# Run the specified queries, with some sanity check.
|
||||
for i in args.queries_to_run:
|
||||
if i < 0 or i >= len(test_queries):
|
||||
print(f'There is no query no. "{i}" in this test, only [{0}-{len(test_queries) - 1}] are present')
|
||||
exit(1)
|
||||
|
||||
queries_to_run = args.queries_to_run
|
||||
|
||||
# Run test queries.
|
||||
for query_index in queries_to_run:
|
||||
|
@ -187,8 +187,10 @@ def td(value, cell_attributes = ''):
|
||||
cell_attributes = cell_attributes,
|
||||
value = value)
|
||||
|
||||
def th(x):
|
||||
return '<th>' + str(x) + '</th>'
|
||||
def th(value, cell_attributes = ''):
|
||||
return '<th {cell_attributes}>{value}</th>'.format(
|
||||
cell_attributes = cell_attributes,
|
||||
value = value)
|
||||
|
||||
def tableRow(cell_values, cell_attributes = [], anchor=None):
|
||||
return tr(
|
||||
@ -199,8 +201,13 @@ def tableRow(cell_values, cell_attributes = [], anchor=None):
|
||||
if a is not None and v is not None]),
|
||||
anchor)
|
||||
|
||||
def tableHeader(r):
|
||||
return tr(''.join([th(f) for f in r]))
|
||||
def tableHeader(cell_values, cell_attributes = []):
|
||||
return tr(
|
||||
''.join([th(v, a)
|
||||
for v, a in itertools.zip_longest(
|
||||
cell_values, cell_attributes,
|
||||
fillvalue = '')
|
||||
if a is not None and v is not None]))
|
||||
|
||||
def tableStart(title):
|
||||
cls = '-'.join(title.lower().split(' ')[:3]);
|
||||
@ -377,16 +384,16 @@ if args.report == 'main':
|
||||
'Ratio of speedup (-) or slowdown (+)', # 2
|
||||
'Relative difference (new − old) / old', # 3
|
||||
'p < 0.01 threshold', # 4
|
||||
# Failed # 5
|
||||
'', # Failed # 5
|
||||
'Test', # 6
|
||||
'#', # 7
|
||||
'Query', # 8
|
||||
]
|
||||
|
||||
text += tableHeader(columns)
|
||||
|
||||
attrs = ['' for c in columns]
|
||||
attrs[5] = None
|
||||
|
||||
text += tableHeader(columns, attrs)
|
||||
|
||||
for row in rows:
|
||||
anchor = f'{currentTableAnchor()}.{row[6]}.{row[7]}'
|
||||
if int(row[5]):
|
||||
@ -421,17 +428,17 @@ if args.report == 'main':
|
||||
'New, s', #1
|
||||
'Relative difference (new - old)/old', #2
|
||||
'p < 0.01 threshold', #3
|
||||
# Failed #4
|
||||
'', # Failed #4
|
||||
'Test', #5
|
||||
'#', #6
|
||||
'Query' #7
|
||||
]
|
||||
|
||||
text = tableStart('Unstable Queries')
|
||||
text += tableHeader(columns)
|
||||
|
||||
attrs = ['' for c in columns]
|
||||
attrs[4] = None
|
||||
|
||||
text = tableStart('Unstable Queries')
|
||||
text += tableHeader(columns, attrs)
|
||||
|
||||
for r in unstable_rows:
|
||||
anchor = f'{currentTableAnchor()}.{r[5]}.{r[6]}'
|
||||
if int(r[4]):
|
||||
@ -464,19 +471,19 @@ if args.report == 'main':
|
||||
'Test', #0
|
||||
'Wall clock time, s', #1
|
||||
'Total client time, s', #2
|
||||
'Total queries', #3
|
||||
'Total queries', #3
|
||||
'Longest query<br>(sum for all runs), s', #4
|
||||
'Avg wall clock time<br>(sum for all runs), s', #5
|
||||
'Shortest query<br>(sum for all runs), s', #6
|
||||
# 'Runs' #7
|
||||
'', # Runs #7
|
||||
]
|
||||
attrs = ['' for c in columns]
|
||||
attrs[7] = None
|
||||
|
||||
text = tableStart('Test Times')
|
||||
text += tableHeader(columns)
|
||||
text += tableHeader(columns, attrs)
|
||||
|
||||
allowed_average_run_time = 3.75 # 60 seconds per test at 7 runs
|
||||
allowed_average_run_time = 1.6 # 30 seconds per test at 7 runs
|
||||
for r in rows:
|
||||
anchor = f'{currentTableAnchor()}.{r[0]}'
|
||||
total_runs = (int(r[7]) + 1) * 2 # one prewarm run, two servers
|
||||
@ -581,8 +588,8 @@ elif args.report == 'all-queries':
|
||||
return
|
||||
|
||||
columns = [
|
||||
# Changed #0
|
||||
# Unstable #1
|
||||
'', # Changed #0
|
||||
'', # Unstable #1
|
||||
'Old, s', #2
|
||||
'New, s', #3
|
||||
'Ratio of speedup (-) or slowdown (+)', #4
|
||||
@ -592,13 +599,13 @@ elif args.report == 'all-queries':
|
||||
'#', #8
|
||||
'Query', #9
|
||||
]
|
||||
|
||||
text = tableStart('All Query Times')
|
||||
text += tableHeader(columns)
|
||||
|
||||
attrs = ['' for c in columns]
|
||||
attrs[0] = None
|
||||
attrs[1] = None
|
||||
|
||||
text = tableStart('All Query Times')
|
||||
text += tableHeader(columns, attrs)
|
||||
|
||||
for r in rows:
|
||||
anchor = f'{currentTableAnchor()}.{r[7]}.{r[8]}'
|
||||
if int(r[1]):
|
||||
|
@ -8,27 +8,8 @@ dpkg -i package_folder/clickhouse-server_*.deb
|
||||
dpkg -i package_folder/clickhouse-client_*.deb
|
||||
dpkg -i package_folder/clickhouse-test_*.deb
|
||||
|
||||
mkdir -p /etc/clickhouse-server/dict_examples
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/zookeeper.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/listen.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/metric_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/readonly.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/macros.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_configd.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_usersd.xml /etc/clickhouse-server/users.d/
|
||||
|
||||
if [[ -n "$USE_DATABASE_ATOMIC" ]] && [[ "$USE_DATABASE_ATOMIC" -eq 1 ]]; then #FIXME USE_DATABASE_ORDINARY
|
||||
ln -s /usr/share/clickhouse-test/config/database_ordinary_usersd.xml /etc/clickhouse-server/config.d/
|
||||
fi
|
||||
# install test configs
|
||||
/usr/share/clickhouse-test/config/install.sh
|
||||
|
||||
function start()
|
||||
{
|
||||
|
@ -48,28 +48,8 @@ mkdir -p /var/lib/clickhouse
|
||||
mkdir -p /var/log/clickhouse-server
|
||||
chmod 777 -R /var/log/clickhouse-server/
|
||||
|
||||
# Temorary way to keep CI green while moving dictionaries to separate directory
|
||||
mkdir -p /etc/clickhouse-server/dict_examples
|
||||
chmod 777 -R /etc/clickhouse-server/dict_examples
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/dict_examples/; \
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/dict_examples/; \
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/dict_examples/;
|
||||
|
||||
ln -s /usr/share/clickhouse-test/config/zookeeper.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/listen.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/metric_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/readonly.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/macros.xml /etc/clickhouse-server/config.d/
|
||||
|
||||
# Retain any pre-existing config and allow ClickHouse to load those if required
|
||||
ln -s --backup=simple --suffix=_original.xml \
|
||||
/usr/share/clickhouse-test/config/query_masking_rules.xml /etc/clickhouse-server/config.d/
|
||||
# install test configs
|
||||
/usr/share/clickhouse-test/config/install.sh
|
||||
|
||||
function start()
|
||||
{
|
||||
|
@ -21,9 +21,7 @@ RUN apt-get update -y \
|
||||
telnet \
|
||||
tree \
|
||||
unixodbc \
|
||||
wget \
|
||||
zookeeper \
|
||||
zookeeperd
|
||||
wget
|
||||
|
||||
RUN mkdir -p /tmp/clickhouse-odbc-tmp \
|
||||
&& wget -nv -O - ${odbc_driver_url} | tar --strip-components=1 -xz -C /tmp/clickhouse-odbc-tmp \
|
||||
|
@ -8,49 +8,9 @@ dpkg -i package_folder/clickhouse-server_*.deb
|
||||
dpkg -i package_folder/clickhouse-client_*.deb
|
||||
dpkg -i package_folder/clickhouse-test_*.deb
|
||||
|
||||
mkdir -p /etc/clickhouse-server/dict_examples
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/zookeeper.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/listen.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/metric_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/custom_settings_prefixes.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/readonly.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/access_management.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/executable_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/macros.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/disks.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/secure_ports.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/clusters.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/graphite.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/server.key /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/server.crt /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/dhparam.pem /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_configd.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_usersd.xml /etc/clickhouse-server/users.d/
|
||||
# install test configs
|
||||
/usr/share/clickhouse-test/config/install.sh
|
||||
|
||||
# Retain any pre-existing config and allow ClickHouse to load it if required
|
||||
ln -s --backup=simple --suffix=_original.xml \
|
||||
/usr/share/clickhouse-test/config/query_masking_rules.xml /etc/clickhouse-server/config.d/
|
||||
|
||||
if [[ -n "$USE_POLYMORPHIC_PARTS" ]] && [[ "$USE_POLYMORPHIC_PARTS" -eq 1 ]]; then
|
||||
ln -s /usr/share/clickhouse-test/config/polymorphic_parts.xml /etc/clickhouse-server/config.d/
|
||||
fi
|
||||
if [[ -n "$USE_DATABASE_ATOMIC" ]] && [[ "$USE_DATABASE_ATOMIC" -eq 1 ]]; then #FIXME USE_DATABASE_ORDINARY
|
||||
ln -s /usr/share/clickhouse-test/config/database_ordinary_usersd.xml /etc/clickhouse-server/users.d/
|
||||
fi
|
||||
|
||||
ln -sf /usr/share/clickhouse-test/config/client_config.xml /etc/clickhouse-client/config.xml
|
||||
|
||||
service zookeeper start
|
||||
sleep 5
|
||||
service clickhouse-server start && sleep 5
|
||||
|
||||
if cat /usr/bin/clickhouse-test | grep -q -- "--use-skip-list"; then
|
||||
|
@ -66,9 +66,7 @@ RUN apt-get --allow-unauthenticated update -y \
|
||||
unixodbc \
|
||||
unixodbc-dev \
|
||||
wget \
|
||||
zlib1g-dev \
|
||||
zookeeper \
|
||||
zookeeperd
|
||||
zlib1g-dev
|
||||
|
||||
RUN mkdir -p /tmp/clickhouse-odbc-tmp \
|
||||
&& wget -nv -O - ${odbc_driver_url} | tar --strip-components=1 -xz -C /tmp/clickhouse-odbc-tmp \
|
||||
|
@ -8,49 +8,9 @@ dpkg -i package_folder/clickhouse-server_*.deb
|
||||
dpkg -i package_folder/clickhouse-client_*.deb
|
||||
dpkg -i package_folder/clickhouse-test_*.deb
|
||||
|
||||
mkdir -p /etc/clickhouse-server/dict_examples
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/dict_examples/
|
||||
ln -s /usr/share/clickhouse-test/config/zookeeper.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/listen.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/metric_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/custom_settings_prefixes.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/readonly.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/access_management.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/executable_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/macros.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/disks.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/secure_ports.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/clusters.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/graphite.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/server.key /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/server.crt /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/dhparam.pem /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_configd.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/database_atomic_usersd.xml /etc/clickhouse-server/users.d/
|
||||
# install test configs
|
||||
/usr/share/clickhouse-test/config/install.sh
|
||||
|
||||
# Retain any pre-existing config and allow ClickHouse to load it if required
|
||||
ln -s --backup=simple --suffix=_original.xml \
|
||||
/usr/share/clickhouse-test/config/query_masking_rules.xml /etc/clickhouse-server/config.d/
|
||||
|
||||
if [[ -n "$USE_POLYMORPHIC_PARTS" ]] && [[ "$USE_POLYMORPHIC_PARTS" -eq 1 ]]; then
|
||||
ln -s /usr/share/clickhouse-test/config/polymorphic_parts.xml /etc/clickhouse-server/config.d/
|
||||
fi
|
||||
if [[ -n "$USE_DATABASE_ATOMIC" ]] && [[ "$USE_DATABASE_ATOMIC" -eq 1 ]]; then #FIXME USE_DATABASE_ORDINARY
|
||||
ln -s /usr/share/clickhouse-test/config/database_ordinary_usersd.xml /etc/clickhouse-server/config.d/
|
||||
fi
|
||||
|
||||
ln -sf /usr/share/clickhouse-test/config/client_config.xml /etc/clickhouse-client/config.xml
|
||||
|
||||
service zookeeper start
|
||||
sleep 5
|
||||
service clickhouse-server start && sleep 5
|
||||
|
||||
if cat /usr/bin/clickhouse-test | grep -q -- "--use-skip-list"; then
|
||||
|
@ -11,8 +11,6 @@ RUN apt-get update -y \
|
||||
tzdata \
|
||||
fakeroot \
|
||||
debhelper \
|
||||
zookeeper \
|
||||
zookeeperd \
|
||||
expect \
|
||||
python \
|
||||
python-lxml \
|
||||
|
@ -39,41 +39,8 @@ mkdir -p /var/log/clickhouse-server
|
||||
chmod 777 -R /var/lib/clickhouse
|
||||
chmod 777 -R /var/log/clickhouse-server/
|
||||
|
||||
# Temorary way to keep CI green while moving dictionaries to separate directory
|
||||
mkdir -p /etc/clickhouse-server/dict_examples
|
||||
chmod 777 -R /etc/clickhouse-server/dict_examples
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/dict_examples/; \
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/dict_examples/; \
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/dict_examples/;
|
||||
|
||||
ln -s /usr/share/clickhouse-test/config/zookeeper.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/listen.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/metric_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/readonly.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/access_management.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/ints_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/strings_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/decimals_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/executable_dictionary.xml /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/macros.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/disks.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/secure_ports.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/clusters.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/graphite.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/server.key /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/server.crt /etc/clickhouse-server/
|
||||
ln -s /usr/share/clickhouse-test/config/dhparam.pem /etc/clickhouse-server/
|
||||
ln -sf /usr/share/clickhouse-test/config/client_config.xml /etc/clickhouse-client/config.xml
|
||||
|
||||
# Retain any pre-existing config and allow ClickHouse to load it if required
|
||||
ln -s --backup=simple --suffix=_original.xml \
|
||||
/usr/share/clickhouse-test/config/query_masking_rules.xml /etc/clickhouse-server/config.d/
|
||||
|
||||
service zookeeper start
|
||||
sleep 5
|
||||
# install test configs
|
||||
/usr/share/clickhouse-test/config/install.sh
|
||||
|
||||
start_clickhouse
|
||||
|
||||
|
@ -39,9 +39,8 @@ function start()
|
||||
done
|
||||
}
|
||||
|
||||
ln -s /usr/share/clickhouse-test/config/log_queries.xml /etc/clickhouse-server/users.d/
|
||||
ln -s /usr/share/clickhouse-test/config/part_log.xml /etc/clickhouse-server/config.d/
|
||||
ln -s /usr/share/clickhouse-test/config/text_log.xml /etc/clickhouse-server/config.d/
|
||||
# install test configs
|
||||
/usr/share/clickhouse-test/config/install.sh
|
||||
|
||||
echo "ASAN_OPTIONS='malloc_context_size=10 verbosity=1 allocator_release_to_os_interval_ms=10000'" >> /etc/environment
|
||||
|
||||
|
@ -35,7 +35,7 @@ RUN apt-get update \
|
||||
ENV TZ=Europe/Moscow
|
||||
RUN ln -snf /usr/share/zoneinfo/$TZ /etc/localtime && echo $TZ > /etc/timezone
|
||||
|
||||
RUN pip3 install urllib3 testflows==1.6.42 docker-compose docker dicttoxml kazoo tzlocal
|
||||
RUN pip3 install urllib3 testflows==1.6.48 docker-compose docker dicttoxml kazoo tzlocal
|
||||
|
||||
ENV DOCKER_CHANNEL stable
|
||||
ENV DOCKER_VERSION 17.09.1-ce
|
||||
|
@ -165,22 +165,6 @@ Similar to GraphiteMergeTree, the Kafka engine supports extended configuration u
|
||||
|
||||
For a list of possible configuration options, see the [librdkafka configuration reference](https://github.com/edenhill/librdkafka/blob/master/CONFIGURATION.md). Use the underscore (`_`) instead of a dot in the ClickHouse configuration. For example, `check.crcs=true` will be `<check_crcs>true</check_crcs>`.
|
||||
|
||||
### Kerberos support {#kafka-kerberos-support}
|
||||
|
||||
To deal with Kerberos-aware Kafka, add `security_protocol` child element with `sasl_plaintext` value. It is enough if Kerberos ticket-granting ticket is obtained and cached by OS facilities.
|
||||
ClickHouse is able to maintain Kerberos credentials using a keytab file. Consider `sasl_kerberos_service_name`, `sasl_kerberos_keytab`, `sasl_kerberos_principal` and `sasl.kerberos.kinit.cmd` child elements.
|
||||
|
||||
Example:
|
||||
|
||||
``` xml
|
||||
<!-- Kerberos-aware Kafka -->
|
||||
<kafka>
|
||||
<security_protocol>SASL_PLAINTEXT</security_protocol>
|
||||
<sasl_kerberos_keytab>/home/kafkauser/kafkauser.keytab</sasl_kerberos_keytab>
|
||||
<sasl_kerberos_principal>kafkauser/kafkahost@EXAMPLE.COM</sasl_kerberos_principal>
|
||||
</kafka>
|
||||
```
|
||||
|
||||
## Virtual Columns {#virtual-columns}
|
||||
|
||||
- `_topic` — Kafka topic.
|
||||
|
@ -9,8 +9,7 @@ toc_title: Working with maps
|
||||
|
||||
Collect all the keys and sum corresponding values.
|
||||
|
||||
Arguments are tuples of two arrays, where items in the first array represent keys, and the second array
|
||||
contains values for the each key.
|
||||
Arguments are tuples of two arrays, where items in the first array represent keys, and the second array contains values for the each key.
|
||||
All key arrays should have same type, and all value arrays should contain items which are promotable to the one type (Int64, UInt64 or Float64).
|
||||
The common promoted type is used as a type for the result array.
|
||||
|
||||
@ -30,8 +29,7 @@ SELECT mapAdd(([toUInt8(1), 2], [1, 1]), ([toUInt8(1), 2], [1, 1])) as res, toTy
|
||||
|
||||
Collect all the keys and subtract corresponding values.
|
||||
|
||||
Arguments are tuples of two arrays, where items in the first array represent keys, and the second array
|
||||
contains values for the each key.
|
||||
Arguments are tuples of two arrays, where items in the first array represent keys, and the second array contains values for the each key.
|
||||
All key arrays should have same type, and all value arrays should contain items which are promotable to the one type (Int64, UInt64 or Float64).
|
||||
The common promoted type is used as a type for the result array.
|
||||
|
||||
@ -45,25 +43,24 @@ SELECT mapSubtract(([toUInt8(1), 2], [toInt32(1), 1]), ([toUInt8(1), 2], [toInt3
|
||||
┌─res────────────┬─type──────────────────────────────┐
|
||||
│ ([1,2],[-1,0]) │ Tuple(Array(UInt8), Array(Int64)) │
|
||||
└────────────────┴───────────────────────────────────┘
|
||||
````
|
||||
```
|
||||
|
||||
## mapPopulateSeries {#function-mappopulateseries}
|
||||
|
||||
Syntax: `mapPopulateSeries((keys : Array(<IntegerType>), values : Array(<IntegerType>)[, max : <IntegerType>])`
|
||||
|
||||
Generates a map, where keys are a series of numbers, from minimum to maximum keys (or `max` argument if it specified) taken from `keys` array with step size of one,
|
||||
and corresponding values taken from `values` array. If the value is not specified for the key, then it uses default value in the resulting map.
|
||||
Generates a map, where keys are a series of numbers, from minimum to maximum keys (or `max` argument if it specified) taken from `keys` array with step size of one, and corresponding values taken from `values` array. If the value is not specified for the key, then it uses default value in the resulting map.
|
||||
For repeated keys only the first value (in order of appearing) gets associated with the key.
|
||||
|
||||
The number of elements in `keys` and `values` must be the same for each row.
|
||||
|
||||
Returns a tuple of two arrays: keys in sorted order, and values the corresponding keys.
|
||||
|
||||
``` sql
|
||||
```sql
|
||||
select mapPopulateSeries([1,2,4], [11,22,44], 5) as res, toTypeName(res) as type;
|
||||
```
|
||||
|
||||
``` text
|
||||
```text
|
||||
┌─res──────────────────────────┬─type──────────────────────────────┐
|
||||
│ ([1,2,3,4,5],[11,22,0,44,0]) │ Tuple(Array(UInt8), Array(UInt8)) │
|
||||
└──────────────────────────────┴───────────────────────────────────┘
|
||||
|
10
release
10
release
@ -95,9 +95,9 @@ then
|
||||
exit 3
|
||||
fi
|
||||
|
||||
export DEB_CC=${DEB_CC=clang-6.0}
|
||||
export DEB_CXX=${DEB_CXX=clang++-6.0}
|
||||
EXTRAPACKAGES="$EXTRAPACKAGES clang-6.0 lld-6.0"
|
||||
export DEB_CC=${DEB_CC=clang-10}
|
||||
export DEB_CXX=${DEB_CXX=clang++-10}
|
||||
EXTRAPACKAGES="$EXTRAPACKAGES clang-10 lld-10"
|
||||
elif [[ $BUILD_TYPE == 'valgrind' ]]; then
|
||||
MALLOC_OPTS="-DENABLE_TCMALLOC=0 -DENABLE_JEMALLOC=0"
|
||||
VERSION_POSTFIX+="+valgrind"
|
||||
@ -118,8 +118,8 @@ echo -e "\nCurrent version is $VERSION_STRING"
|
||||
if [ -z "$NO_BUILD" ] ; then
|
||||
gen_changelog "$VERSION_STRING" "" "$AUTHOR" ""
|
||||
if [ -z "$USE_PBUILDER" ] ; then
|
||||
DEB_CC=${DEB_CC:=`which gcc-9 gcc-8 gcc | head -n1`}
|
||||
DEB_CXX=${DEB_CXX:=`which g++-9 g++-8 g++ | head -n1`}
|
||||
DEB_CC=${DEB_CC:=`which gcc-10 gcc-9 gcc | head -n1`}
|
||||
DEB_CXX=${DEB_CXX:=`which gcc-10 g++-9 g++ | head -n1`}
|
||||
# Build (only binary packages).
|
||||
debuild --preserve-env -e PATH \
|
||||
-e DEB_CC=$DEB_CC -e DEB_CXX=$DEB_CXX -e CMAKE_FLAGS="$CMAKE_FLAGS" \
|
||||
|
@ -192,7 +192,7 @@ namespace
|
||||
}
|
||||
|
||||
|
||||
std::vector<AccessEntityPtr> parseUsers(const Poco::Util::AbstractConfiguration & config, Poco::Logger * log)
|
||||
std::vector<AccessEntityPtr> parseUsers(const Poco::Util::AbstractConfiguration & config)
|
||||
{
|
||||
Poco::Util::AbstractConfiguration::Keys user_names;
|
||||
config.keys("users", user_names);
|
||||
@ -200,16 +200,8 @@ namespace
|
||||
std::vector<AccessEntityPtr> users;
|
||||
users.reserve(user_names.size());
|
||||
for (const auto & user_name : user_names)
|
||||
{
|
||||
try
|
||||
{
|
||||
users.push_back(parseUser(config, user_name));
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
tryLogCurrentException(log, "Could not parse user " + backQuote(user_name));
|
||||
}
|
||||
}
|
||||
users.push_back(parseUser(config, user_name));
|
||||
|
||||
return users;
|
||||
}
|
||||
|
||||
@ -256,12 +248,11 @@ namespace
|
||||
}
|
||||
|
||||
quota->to_roles.add(user_ids);
|
||||
|
||||
return quota;
|
||||
}
|
||||
|
||||
|
||||
std::vector<AccessEntityPtr> parseQuotas(const Poco::Util::AbstractConfiguration & config, Poco::Logger * log)
|
||||
std::vector<AccessEntityPtr> parseQuotas(const Poco::Util::AbstractConfiguration & config)
|
||||
{
|
||||
Poco::Util::AbstractConfiguration::Keys user_names;
|
||||
config.keys("users", user_names);
|
||||
@ -278,76 +269,63 @@ namespace
|
||||
quotas.reserve(quota_names.size());
|
||||
for (const auto & quota_name : quota_names)
|
||||
{
|
||||
try
|
||||
{
|
||||
auto it = quota_to_user_ids.find(quota_name);
|
||||
const std::vector<UUID> & quota_users = (it != quota_to_user_ids.end()) ? std::move(it->second) : std::vector<UUID>{};
|
||||
quotas.push_back(parseQuota(config, quota_name, quota_users));
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
tryLogCurrentException(log, "Could not parse quota " + backQuote(quota_name));
|
||||
}
|
||||
auto it = quota_to_user_ids.find(quota_name);
|
||||
const std::vector<UUID> & quota_users = (it != quota_to_user_ids.end()) ? std::move(it->second) : std::vector<UUID>{};
|
||||
quotas.push_back(parseQuota(config, quota_name, quota_users));
|
||||
}
|
||||
return quotas;
|
||||
}
|
||||
|
||||
|
||||
std::vector<AccessEntityPtr> parseRowPolicies(const Poco::Util::AbstractConfiguration & config, Poco::Logger * log)
|
||||
std::vector<AccessEntityPtr> parseRowPolicies(const Poco::Util::AbstractConfiguration & config)
|
||||
{
|
||||
std::map<std::pair<String /* database */, String /* table */>, std::unordered_map<String /* user */, String /* filter */>> all_filters_map;
|
||||
|
||||
Poco::Util::AbstractConfiguration::Keys user_names;
|
||||
config.keys("users", user_names);
|
||||
|
||||
try
|
||||
for (const String & user_name : user_names)
|
||||
{
|
||||
config.keys("users", user_names);
|
||||
for (const String & user_name : user_names)
|
||||
const String databases_config = "users." + user_name + ".databases";
|
||||
if (config.has(databases_config))
|
||||
{
|
||||
const String databases_config = "users." + user_name + ".databases";
|
||||
if (config.has(databases_config))
|
||||
Poco::Util::AbstractConfiguration::Keys database_keys;
|
||||
config.keys(databases_config, database_keys);
|
||||
|
||||
/// Read tables within databases
|
||||
for (const String & database_key : database_keys)
|
||||
{
|
||||
Poco::Util::AbstractConfiguration::Keys database_keys;
|
||||
config.keys(databases_config, database_keys);
|
||||
const String database_config = databases_config + "." + database_key;
|
||||
|
||||
/// Read tables within databases
|
||||
for (const String & database_key : database_keys)
|
||||
String database_name;
|
||||
if (((database_key == "database") || (database_key.starts_with("database["))) && config.has(database_config + "[@name]"))
|
||||
database_name = config.getString(database_config + "[@name]");
|
||||
else if (size_t bracket_pos = database_key.find('['); bracket_pos != std::string::npos)
|
||||
database_name = database_key.substr(0, bracket_pos);
|
||||
else
|
||||
database_name = database_key;
|
||||
|
||||
Poco::Util::AbstractConfiguration::Keys table_keys;
|
||||
config.keys(database_config, table_keys);
|
||||
|
||||
/// Read table properties
|
||||
for (const String & table_key : table_keys)
|
||||
{
|
||||
const String database_config = databases_config + "." + database_key;
|
||||
|
||||
String database_name;
|
||||
if (((database_key == "database") || (database_key.starts_with("database["))) && config.has(database_config + "[@name]"))
|
||||
database_name = config.getString(database_config + "[@name]");
|
||||
else if (size_t bracket_pos = database_key.find('['); bracket_pos != std::string::npos)
|
||||
database_name = database_key.substr(0, bracket_pos);
|
||||
String table_config = database_config + "." + table_key;
|
||||
String table_name;
|
||||
if (((table_key == "table") || (table_key.starts_with("table["))) && config.has(table_config + "[@name]"))
|
||||
table_name = config.getString(table_config + "[@name]");
|
||||
else if (size_t bracket_pos = table_key.find('['); bracket_pos != std::string::npos)
|
||||
table_name = table_key.substr(0, bracket_pos);
|
||||
else
|
||||
database_name = database_key;
|
||||
table_name = table_key;
|
||||
|
||||
Poco::Util::AbstractConfiguration::Keys table_keys;
|
||||
config.keys(database_config, table_keys);
|
||||
|
||||
/// Read table properties
|
||||
for (const String & table_key : table_keys)
|
||||
{
|
||||
String table_config = database_config + "." + table_key;
|
||||
String table_name;
|
||||
if (((table_key == "table") || (table_key.starts_with("table["))) && config.has(table_config + "[@name]"))
|
||||
table_name = config.getString(table_config + "[@name]");
|
||||
else if (size_t bracket_pos = table_key.find('['); bracket_pos != std::string::npos)
|
||||
table_name = table_key.substr(0, bracket_pos);
|
||||
else
|
||||
table_name = table_key;
|
||||
|
||||
String filter_config = table_config + ".filter";
|
||||
all_filters_map[{database_name, table_name}][user_name] = config.getString(filter_config);
|
||||
}
|
||||
String filter_config = table_config + ".filter";
|
||||
all_filters_map[{database_name, table_name}][user_name] = config.getString(filter_config);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
tryLogCurrentException(log, "Could not parse row policies");
|
||||
}
|
||||
|
||||
std::vector<AccessEntityPtr> policies;
|
||||
for (auto & [database_and_table_name, user_to_filters] : all_filters_map)
|
||||
@ -450,23 +428,14 @@ namespace
|
||||
|
||||
std::vector<AccessEntityPtr> parseSettingsProfiles(
|
||||
const Poco::Util::AbstractConfiguration & config,
|
||||
const std::function<void(const std::string_view &)> & check_setting_name_function,
|
||||
Poco::Logger * log)
|
||||
const std::function<void(const std::string_view &)> & check_setting_name_function)
|
||||
{
|
||||
std::vector<AccessEntityPtr> profiles;
|
||||
Poco::Util::AbstractConfiguration::Keys profile_names;
|
||||
config.keys("profiles", profile_names);
|
||||
for (const auto & profile_name : profile_names)
|
||||
{
|
||||
try
|
||||
{
|
||||
profiles.push_back(parseSettingsProfile(config, profile_name, check_setting_name_function));
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
tryLogCurrentException(log, "Could not parse profile " + backQuote(profile_name));
|
||||
}
|
||||
}
|
||||
profiles.push_back(parseSettingsProfile(config, profile_name, check_setting_name_function));
|
||||
|
||||
return profiles;
|
||||
}
|
||||
}
|
||||
@ -520,13 +489,13 @@ void UsersConfigAccessStorage::setConfig(const Poco::Util::AbstractConfiguration
|
||||
void UsersConfigAccessStorage::parseFromConfig(const Poco::Util::AbstractConfiguration & config)
|
||||
{
|
||||
std::vector<std::pair<UUID, AccessEntityPtr>> all_entities;
|
||||
for (const auto & entity : parseUsers(config, getLogger()))
|
||||
for (const auto & entity : parseUsers(config))
|
||||
all_entities.emplace_back(generateID(*entity), entity);
|
||||
for (const auto & entity : parseQuotas(config, getLogger()))
|
||||
for (const auto & entity : parseQuotas(config))
|
||||
all_entities.emplace_back(generateID(*entity), entity);
|
||||
for (const auto & entity : parseRowPolicies(config, getLogger()))
|
||||
for (const auto & entity : parseRowPolicies(config))
|
||||
all_entities.emplace_back(generateID(*entity), entity);
|
||||
for (const auto & entity : parseSettingsProfiles(config, check_setting_name_function, getLogger()))
|
||||
for (const auto & entity : parseSettingsProfiles(config, check_setting_name_function))
|
||||
all_entities.emplace_back(generateID(*entity), entity);
|
||||
memory_storage.setAll(all_entities);
|
||||
}
|
||||
|
@ -138,6 +138,7 @@ void ConfigReloader::reloadIfNewer(bool force, bool throw_on_error, bool fallbac
|
||||
if (throw_on_error)
|
||||
throw;
|
||||
tryLogCurrentException(log, "Error updating configuration from '" + path + "' config.");
|
||||
return;
|
||||
}
|
||||
|
||||
LOG_DEBUG(log, "Loaded config '{}', performed update on configuration", path);
|
||||
|
@ -41,6 +41,11 @@ void FileChecker::setEmpty(const String & full_file_path)
|
||||
map[fileName(full_file_path)] = 0;
|
||||
}
|
||||
|
||||
FileChecker::Map FileChecker::getFileSizes() const
|
||||
{
|
||||
return map;
|
||||
}
|
||||
|
||||
CheckResults FileChecker::check() const
|
||||
{
|
||||
// Read the files again every time you call `check` - so as not to violate the constancy.
|
||||
|
@ -27,10 +27,12 @@ public:
|
||||
/// The purpose of this function is to rollback a group of unfinished writes.
|
||||
void repair();
|
||||
|
||||
private:
|
||||
/// File name -> size.
|
||||
using Map = std::map<String, UInt64>;
|
||||
|
||||
Map getFileSizes() const;
|
||||
|
||||
private:
|
||||
void initialize();
|
||||
void updateImpl(const String & file_path);
|
||||
void load(Map & local_map, const String & path) const;
|
||||
|
@ -4,6 +4,7 @@
|
||||
#include <Common/Exception.h>
|
||||
#include <Common/randomSeed.h>
|
||||
#include <Common/SipHash.h>
|
||||
#include <common/getThreadId.h>
|
||||
#include <common/types.h>
|
||||
|
||||
|
||||
@ -19,7 +20,7 @@ namespace DB
|
||||
DB::UInt64 randomSeed()
|
||||
{
|
||||
struct timespec times;
|
||||
if (clock_gettime(CLOCK_THREAD_CPUTIME_ID, ×))
|
||||
if (clock_gettime(CLOCK_MONOTONIC, ×))
|
||||
DB::throwFromErrno("Cannot clock_gettime.", DB::ErrorCodes::CANNOT_CLOCK_GETTIME);
|
||||
|
||||
/// Not cryptographically secure as time, pid and stack address can be predictable.
|
||||
@ -27,7 +28,7 @@ DB::UInt64 randomSeed()
|
||||
SipHash hash;
|
||||
hash.update(times.tv_nsec);
|
||||
hash.update(times.tv_sec);
|
||||
hash.update(getpid());
|
||||
hash.update(getThreadId());
|
||||
hash.update(×);
|
||||
return hash.get64();
|
||||
}
|
||||
|
@ -77,7 +77,7 @@ ColumnDefinition::ColumnDefinition(
|
||||
|
||||
size_t ColumnDefinition::getPayloadSize() const
|
||||
{
|
||||
return 13 + getLengthEncodedStringSize("def") + getLengthEncodedStringSize(schema) + getLengthEncodedStringSize(table) + getLengthEncodedStringSize(org_table) + \
|
||||
return 12 + getLengthEncodedStringSize("def") + getLengthEncodedStringSize(schema) + getLengthEncodedStringSize(table) + getLengthEncodedStringSize(org_table) + \
|
||||
getLengthEncodedStringSize(name) + getLengthEncodedStringSize(org_name) + getLengthEncodedNumberSize(next_length);
|
||||
}
|
||||
|
||||
@ -96,7 +96,7 @@ void ColumnDefinition::readPayloadImpl(ReadBuffer & payload)
|
||||
payload.readStrict(reinterpret_cast<char *>(&column_length), 4);
|
||||
payload.readStrict(reinterpret_cast<char *>(&column_type), 1);
|
||||
payload.readStrict(reinterpret_cast<char *>(&flags), 2);
|
||||
payload.readStrict(reinterpret_cast<char *>(&decimals), 2);
|
||||
payload.readStrict(reinterpret_cast<char *>(&decimals), 1);
|
||||
payload.ignore(2);
|
||||
}
|
||||
|
||||
@ -113,7 +113,7 @@ void ColumnDefinition::writePayloadImpl(WriteBuffer & buffer) const
|
||||
buffer.write(reinterpret_cast<const char *>(&column_length), 4);
|
||||
buffer.write(reinterpret_cast<const char *>(&column_type), 1);
|
||||
buffer.write(reinterpret_cast<const char *>(&flags), 2);
|
||||
buffer.write(reinterpret_cast<const char *>(&decimals), 2);
|
||||
buffer.write(reinterpret_cast<const char *>(&decimals), 1);
|
||||
writeChar(0x0, 2, buffer);
|
||||
}
|
||||
|
||||
|
@ -926,7 +926,7 @@ void CacheDictionary::update(UpdateUnitPtr & update_unit_ptr) const
|
||||
else
|
||||
cell.setExpiresAt(std::chrono::time_point<std::chrono::system_clock>::max());
|
||||
|
||||
update_unit_ptr->getPresentIdHandler()(id, cell_idx);
|
||||
update_unit_ptr->callPresentIdHandler(id, cell_idx);
|
||||
/// mark corresponding id as found
|
||||
remaining_ids[id] = 1;
|
||||
}
|
||||
@ -988,9 +988,9 @@ void CacheDictionary::update(UpdateUnitPtr & update_unit_ptr) const
|
||||
if (was_default)
|
||||
cell.setDefault();
|
||||
if (was_default)
|
||||
update_unit_ptr->getAbsentIdHandler()(id, cell_idx);
|
||||
update_unit_ptr->callAbsentIdHandler(id, cell_idx);
|
||||
else
|
||||
update_unit_ptr->getPresentIdHandler()(id, cell_idx);
|
||||
update_unit_ptr->callPresentIdHandler(id, cell_idx);
|
||||
continue;
|
||||
}
|
||||
/// We don't have expired data for that `id` so all we can do is to rethrow `last_exception`.
|
||||
@ -1022,7 +1022,7 @@ void CacheDictionary::update(UpdateUnitPtr & update_unit_ptr) const
|
||||
setDefaultAttributeValue(attribute, cell_idx);
|
||||
|
||||
/// inform caller that the cell has not been found
|
||||
update_unit_ptr->getAbsentIdHandler()(id, cell_idx);
|
||||
update_unit_ptr->callAbsentIdHandler(id, cell_idx);
|
||||
}
|
||||
|
||||
ProfileEvents::increment(ProfileEvents::DictCacheKeysRequestedMiss, not_found_num);
|
||||
|
@ -399,16 +399,18 @@ private:
|
||||
absent_id_handler([](Key, size_t){}){}
|
||||
|
||||
|
||||
PresentIdHandler getPresentIdHandler()
|
||||
void callPresentIdHandler(Key key, size_t cell_idx)
|
||||
{
|
||||
std::lock_guard lock(callback_mutex);
|
||||
return can_use_callback ? present_id_handler : PresentIdHandler{};
|
||||
if (can_use_callback)
|
||||
present_id_handler(key, cell_idx);
|
||||
}
|
||||
|
||||
AbsentIdHandler getAbsentIdHandler()
|
||||
void callAbsentIdHandler(Key key, size_t cell_idx)
|
||||
{
|
||||
std::lock_guard lock(callback_mutex);
|
||||
return can_use_callback ? absent_id_handler : AbsentIdHandler{};
|
||||
if (can_use_callback)
|
||||
absent_id_handler(key, cell_idx);
|
||||
}
|
||||
|
||||
std::vector<Key> requested_ids;
|
||||
|
@ -148,7 +148,9 @@ void CacheDictionary::getItemsNumberImpl(
|
||||
std::begin(cache_expired_ids), std::end(cache_expired_ids),
|
||||
std::back_inserter(required_ids), [](auto & pair) { return pair.first; });
|
||||
|
||||
auto on_cell_updated = [&] (const auto id, const auto cell_idx)
|
||||
auto on_cell_updated =
|
||||
[&attribute_array, &cache_not_found_ids, &cache_expired_ids, &out]
|
||||
(const auto id, const auto cell_idx)
|
||||
{
|
||||
const auto attribute_value = attribute_array[cell_idx];
|
||||
|
||||
|
@ -119,13 +119,17 @@ struct QueryPlanSettings
|
||||
{
|
||||
QueryPlan::ExplainPlanOptions query_plan_options;
|
||||
|
||||
/// Apply query plan optimisations.
|
||||
bool optimize = true;
|
||||
|
||||
constexpr static char name[] = "PLAN";
|
||||
|
||||
std::unordered_map<std::string, std::reference_wrapper<bool>> boolean_settings =
|
||||
{
|
||||
{"header", query_plan_options.header},
|
||||
{"description", query_plan_options.description},
|
||||
{"actions", query_plan_options.actions}
|
||||
{"actions", query_plan_options.actions},
|
||||
{"optimize", optimize},
|
||||
};
|
||||
};
|
||||
|
||||
@ -248,7 +252,8 @@ BlockInputStreamPtr InterpreterExplainQuery::executeImpl()
|
||||
InterpreterSelectWithUnionQuery interpreter(ast.getExplainedQuery(), context, SelectQueryOptions());
|
||||
interpreter.buildQueryPlan(plan);
|
||||
|
||||
plan.optimize();
|
||||
if (settings.optimize)
|
||||
plan.optimize();
|
||||
|
||||
WriteBufferFromOStream buffer(ss);
|
||||
plan.explainPlan(buffer, settings.query_plan_options);
|
||||
|
@ -547,7 +547,7 @@ bool StorageBuffer::optimize(
|
||||
if (deduplicate)
|
||||
throw Exception("DEDUPLICATE cannot be specified when optimizing table of type Buffer", ErrorCodes::NOT_IMPLEMENTED);
|
||||
|
||||
flushAllBuffers(false);
|
||||
flushAllBuffers(false, true);
|
||||
return true;
|
||||
}
|
||||
|
||||
@ -595,14 +595,14 @@ bool StorageBuffer::checkThresholdsImpl(size_t rows, size_t bytes, time_t time_p
|
||||
}
|
||||
|
||||
|
||||
void StorageBuffer::flushAllBuffers(const bool check_thresholds)
|
||||
void StorageBuffer::flushAllBuffers(bool check_thresholds, bool reset_blocks_structure)
|
||||
{
|
||||
for (auto & buf : buffers)
|
||||
flushBuffer(buf, check_thresholds);
|
||||
flushBuffer(buf, check_thresholds, false, reset_blocks_structure);
|
||||
}
|
||||
|
||||
|
||||
void StorageBuffer::flushBuffer(Buffer & buffer, bool check_thresholds, bool locked)
|
||||
void StorageBuffer::flushBuffer(Buffer & buffer, bool check_thresholds, bool locked, bool reset_block_structure)
|
||||
{
|
||||
Block block_to_write;
|
||||
time_t current_time = time(nullptr);
|
||||
@ -655,6 +655,8 @@ void StorageBuffer::flushBuffer(Buffer & buffer, bool check_thresholds, bool loc
|
||||
try
|
||||
{
|
||||
writeBlockToDestination(block_to_write, DatabaseCatalog::instance().tryGetTable(destination_id, global_context));
|
||||
if (reset_block_structure)
|
||||
buffer.data.clear();
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
@ -829,7 +831,9 @@ void StorageBuffer::alter(const AlterCommands & params, const Context & context,
|
||||
checkAlterIsPossible(params, context.getSettingsRef());
|
||||
auto metadata_snapshot = getInMemoryMetadataPtr();
|
||||
|
||||
/// So that no blocks of the old structure remain.
|
||||
/// Flush all buffers to storages, so that no non-empty blocks of the old
|
||||
/// structure remain. Structure of empty blocks will be updated during first
|
||||
/// insert.
|
||||
optimize({} /*query*/, metadata_snapshot, {} /*partition_id*/, false /*final*/, false /*deduplicate*/, context);
|
||||
|
||||
StorageInMemoryMetadata new_metadata = *metadata_snapshot;
|
||||
|
@ -130,9 +130,11 @@ private:
|
||||
|
||||
Poco::Logger * log;
|
||||
|
||||
void flushAllBuffers(bool check_thresholds = true);
|
||||
/// Reset the buffer. If check_thresholds is set - resets only if thresholds are exceeded.
|
||||
void flushBuffer(Buffer & buffer, bool check_thresholds, bool locked = false);
|
||||
void flushAllBuffers(bool check_thresholds = true, bool reset_blocks_structure = false);
|
||||
/// Reset the buffer. If check_thresholds is set - resets only if thresholds
|
||||
/// are exceeded. If reset_block_structure is set - clears inner block
|
||||
/// structure inside buffer (useful in OPTIMIZE and ALTER).
|
||||
void flushBuffer(Buffer & buffer, bool check_thresholds, bool locked = false, bool reset_block_structure = false);
|
||||
bool checkThresholds(const Buffer & buffer, time_t current_time, size_t additional_rows = 0, size_t additional_bytes = 0) const;
|
||||
bool checkThresholdsImpl(size_t rows, size_t bytes, time_t time_passed) const;
|
||||
|
||||
|
@ -52,6 +52,7 @@ namespace ErrorCodes
|
||||
extern const int UNKNOWN_IDENTIFIER;
|
||||
extern const int INCORRECT_FILE_NAME;
|
||||
extern const int FILE_DOESNT_EXIST;
|
||||
extern const int TIMEOUT_EXCEEDED;
|
||||
}
|
||||
|
||||
namespace
|
||||
@ -199,6 +200,17 @@ StorageFile::StorageFile(CommonArguments args)
|
||||
setInMemoryMetadata(storage_metadata);
|
||||
}
|
||||
|
||||
|
||||
static std::chrono::seconds getLockTimeout(const Context & context)
|
||||
{
|
||||
const Settings & settings = context.getSettingsRef();
|
||||
Int64 lock_timeout = settings.lock_acquire_timeout.totalSeconds();
|
||||
if (settings.max_execution_time.totalSeconds() != 0 && settings.max_execution_time.totalSeconds() < lock_timeout)
|
||||
lock_timeout = settings.max_execution_time.totalSeconds();
|
||||
return std::chrono::seconds{lock_timeout};
|
||||
}
|
||||
|
||||
|
||||
class StorageFileSource : public SourceWithProgress
|
||||
{
|
||||
public:
|
||||
@ -245,7 +257,9 @@ public:
|
||||
{
|
||||
if (storage->use_table_fd)
|
||||
{
|
||||
unique_lock = std::unique_lock(storage->rwlock);
|
||||
unique_lock = std::unique_lock(storage->rwlock, getLockTimeout(context));
|
||||
if (!unique_lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
/// We could use common ReadBuffer and WriteBuffer in storage to leverage cache
|
||||
/// and add ability to seek unseekable files, but cache sync isn't supported.
|
||||
@ -264,7 +278,9 @@ public:
|
||||
}
|
||||
else
|
||||
{
|
||||
shared_lock = std::shared_lock(storage->rwlock);
|
||||
shared_lock = std::shared_lock(storage->rwlock, getLockTimeout(context));
|
||||
if (!shared_lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
}
|
||||
}
|
||||
|
||||
@ -373,8 +389,8 @@ private:
|
||||
|
||||
bool finished_generate = false;
|
||||
|
||||
std::shared_lock<std::shared_mutex> shared_lock;
|
||||
std::unique_lock<std::shared_mutex> unique_lock;
|
||||
std::shared_lock<std::shared_timed_mutex> shared_lock;
|
||||
std::unique_lock<std::shared_timed_mutex> unique_lock;
|
||||
};
|
||||
|
||||
|
||||
@ -417,7 +433,7 @@ Pipe StorageFile::read(
|
||||
|
||||
for (size_t i = 0; i < num_streams; ++i)
|
||||
pipes.emplace_back(std::make_shared<StorageFileSource>(
|
||||
this_ptr, metadata_snapshot, context, max_block_size, files_info, metadata_snapshot->getColumns().getDefaults()));
|
||||
this_ptr, metadata_snapshot, context, max_block_size, files_info, metadata_snapshot->getColumns().getDefaults()));
|
||||
|
||||
return Pipe::unitePipes(std::move(pipes));
|
||||
}
|
||||
@ -429,12 +445,16 @@ public:
|
||||
explicit StorageFileBlockOutputStream(
|
||||
StorageFile & storage_,
|
||||
const StorageMetadataPtr & metadata_snapshot_,
|
||||
std::unique_lock<std::shared_timed_mutex> && lock_,
|
||||
const CompressionMethod compression_method,
|
||||
const Context & context)
|
||||
: storage(storage_)
|
||||
, metadata_snapshot(metadata_snapshot_)
|
||||
, lock(storage.rwlock)
|
||||
, lock(std::move(lock_))
|
||||
{
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
std::unique_ptr<WriteBufferFromFileDescriptor> naked_buffer = nullptr;
|
||||
if (storage.use_table_fd)
|
||||
{
|
||||
@ -488,7 +508,7 @@ public:
|
||||
private:
|
||||
StorageFile & storage;
|
||||
StorageMetadataPtr metadata_snapshot;
|
||||
std::unique_lock<std::shared_mutex> lock;
|
||||
std::unique_lock<std::shared_timed_mutex> lock;
|
||||
std::unique_ptr<WriteBuffer> write_buf;
|
||||
BlockOutputStreamPtr writer;
|
||||
bool prefix_written{false};
|
||||
@ -506,7 +526,7 @@ BlockOutputStreamPtr StorageFile::write(
|
||||
if (!paths.empty())
|
||||
path = paths[0];
|
||||
|
||||
return std::make_shared<StorageFileBlockOutputStream>(*this, metadata_snapshot,
|
||||
return std::make_shared<StorageFileBlockOutputStream>(*this, metadata_snapshot, std::unique_lock{rwlock, getLockTimeout(context)},
|
||||
chooseCompressionMethod(path, compression_method), context);
|
||||
}
|
||||
|
||||
@ -529,8 +549,6 @@ void StorageFile::rename(const String & new_path_to_table_data, const StorageID
|
||||
if (path_new == paths[0])
|
||||
return;
|
||||
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
Poco::File(Poco::Path(path_new).parent()).createDirectories();
|
||||
Poco::File(paths[0]).renameTo(path_new);
|
||||
|
||||
@ -547,8 +565,6 @@ void StorageFile::truncate(
|
||||
if (paths.size() != 1)
|
||||
throw Exception("Can't truncate table '" + getStorageID().getNameForLogs() + "' in readonly mode", ErrorCodes::DATABASE_ACCESS_DENIED);
|
||||
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
if (use_table_fd)
|
||||
{
|
||||
if (0 != ::ftruncate(table_fd, 0))
|
||||
|
@ -89,7 +89,7 @@ private:
|
||||
std::atomic<bool> table_fd_was_used{false}; /// To detect repeating reads from stdin
|
||||
off_t table_fd_init_offset = -1; /// Initial position of fd, used for repeating reads
|
||||
|
||||
mutable std::shared_mutex rwlock;
|
||||
mutable std::shared_timed_mutex rwlock;
|
||||
|
||||
Poco::Logger * log = &Poco::Logger::get("StorageFile");
|
||||
};
|
||||
|
@ -39,6 +39,7 @@ namespace DB
|
||||
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int TIMEOUT_EXCEEDED;
|
||||
extern const int LOGICAL_ERROR;
|
||||
extern const int DUPLICATE_COLUMN;
|
||||
extern const int SIZES_OF_MARKS_FILES_ARE_INCONSISTENT;
|
||||
@ -50,7 +51,6 @@ namespace ErrorCodes
|
||||
class LogSource final : public SourceWithProgress
|
||||
{
|
||||
public:
|
||||
|
||||
static Block getHeader(const NamesAndTypesList & columns)
|
||||
{
|
||||
Block res;
|
||||
@ -116,13 +116,16 @@ private:
|
||||
class LogBlockOutputStream final : public IBlockOutputStream
|
||||
{
|
||||
public:
|
||||
explicit LogBlockOutputStream(StorageLog & storage_, const StorageMetadataPtr & metadata_snapshot_)
|
||||
explicit LogBlockOutputStream(
|
||||
StorageLog & storage_, const StorageMetadataPtr & metadata_snapshot_, std::unique_lock<std::shared_timed_mutex> && lock_)
|
||||
: storage(storage_)
|
||||
, metadata_snapshot(metadata_snapshot_)
|
||||
, lock(storage.rwlock)
|
||||
, lock(std::move(lock_))
|
||||
, marks_stream(
|
||||
storage.disk->writeFile(storage.marks_file_path, 4096, WriteMode::Rewrite))
|
||||
{
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
}
|
||||
|
||||
~LogBlockOutputStream() override
|
||||
@ -149,7 +152,7 @@ public:
|
||||
private:
|
||||
StorageLog & storage;
|
||||
StorageMetadataPtr metadata_snapshot;
|
||||
std::unique_lock<std::shared_mutex> lock;
|
||||
std::unique_lock<std::shared_timed_mutex> lock;
|
||||
bool done = false;
|
||||
|
||||
struct Stream
|
||||
@ -507,9 +510,11 @@ void StorageLog::addFiles(const String & column_name, const IDataType & type)
|
||||
}
|
||||
|
||||
|
||||
void StorageLog::loadMarks()
|
||||
void StorageLog::loadMarks(std::chrono::seconds lock_timeout)
|
||||
{
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
std::unique_lock lock(rwlock, lock_timeout);
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
if (loaded_marks)
|
||||
return;
|
||||
@ -552,8 +557,6 @@ void StorageLog::rename(const String & new_path_to_table_data, const StorageID &
|
||||
{
|
||||
assert(table_path != new_path_to_table_data);
|
||||
{
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
disk->moveDirectory(table_path, new_path_to_table_data);
|
||||
|
||||
table_path = new_path_to_table_data;
|
||||
@ -569,8 +572,6 @@ void StorageLog::rename(const String & new_path_to_table_data, const StorageID &
|
||||
|
||||
void StorageLog::truncate(const ASTPtr &, const StorageMetadataPtr & metadata_snapshot, const Context &, TableExclusiveLockHolder &)
|
||||
{
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
files.clear();
|
||||
file_count = 0;
|
||||
loaded_marks = false;
|
||||
@ -610,6 +611,17 @@ const StorageLog::Marks & StorageLog::getMarksWithRealRowCount(const StorageMeta
|
||||
return it->second.marks;
|
||||
}
|
||||
|
||||
|
||||
static std::chrono::seconds getLockTimeout(const Context & context)
|
||||
{
|
||||
const Settings & settings = context.getSettingsRef();
|
||||
Int64 lock_timeout = settings.lock_acquire_timeout.totalSeconds();
|
||||
if (settings.max_execution_time.totalSeconds() != 0 && settings.max_execution_time.totalSeconds() < lock_timeout)
|
||||
lock_timeout = settings.max_execution_time.totalSeconds();
|
||||
return std::chrono::seconds{lock_timeout};
|
||||
}
|
||||
|
||||
|
||||
Pipe StorageLog::read(
|
||||
const Names & column_names,
|
||||
const StorageMetadataPtr & metadata_snapshot,
|
||||
@ -620,11 +632,15 @@ Pipe StorageLog::read(
|
||||
unsigned num_streams)
|
||||
{
|
||||
metadata_snapshot->check(column_names, getVirtuals(), getStorageID());
|
||||
loadMarks();
|
||||
|
||||
auto lock_timeout = getLockTimeout(context);
|
||||
loadMarks(lock_timeout);
|
||||
|
||||
NamesAndTypesList all_columns = Nested::collect(metadata_snapshot->getColumns().getAllPhysical().addTypes(column_names));
|
||||
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
std::shared_lock lock(rwlock, lock_timeout);
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
Pipes pipes;
|
||||
|
||||
@ -653,18 +669,28 @@ Pipe StorageLog::read(
|
||||
max_read_buffer_size));
|
||||
}
|
||||
|
||||
/// No need to hold lock while reading because we read fixed range of data that does not change while appending more data.
|
||||
return Pipe::unitePipes(std::move(pipes));
|
||||
}
|
||||
|
||||
BlockOutputStreamPtr StorageLog::write(const ASTPtr & /*query*/, const StorageMetadataPtr & metadata_snapshot, const Context & /*context*/)
|
||||
BlockOutputStreamPtr StorageLog::write(const ASTPtr & /*query*/, const StorageMetadataPtr & metadata_snapshot, const Context & context)
|
||||
{
|
||||
loadMarks();
|
||||
return std::make_shared<LogBlockOutputStream>(*this, metadata_snapshot);
|
||||
auto lock_timeout = getLockTimeout(context);
|
||||
loadMarks(lock_timeout);
|
||||
|
||||
std::unique_lock lock(rwlock, lock_timeout);
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
return std::make_shared<LogBlockOutputStream>(*this, metadata_snapshot, std::move(lock));
|
||||
}
|
||||
|
||||
CheckResults StorageLog::checkData(const ASTPtr & /* query */, const Context & /* context */)
|
||||
CheckResults StorageLog::checkData(const ASTPtr & /* query */, const Context & context)
|
||||
{
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
std::shared_lock lock(rwlock, getLockTimeout(context));
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
return file_checker.check();
|
||||
}
|
||||
|
||||
|
@ -83,7 +83,7 @@ private:
|
||||
DiskPtr disk;
|
||||
String table_path;
|
||||
|
||||
mutable std::shared_mutex rwlock;
|
||||
mutable std::shared_timed_mutex rwlock;
|
||||
|
||||
Files files;
|
||||
|
||||
@ -104,7 +104,7 @@ private:
|
||||
/// Read marks files if they are not already read.
|
||||
/// It is done lazily, so that with a large number of tables, the server starts quickly.
|
||||
/// You can not call with a write locked `rwlock`.
|
||||
void loadMarks();
|
||||
void loadMarks(std::chrono::seconds lock_timeout);
|
||||
|
||||
/** For normal columns, the number of rows in the block is specified in the marks.
|
||||
* For array columns and nested structures, there are more than one group of marks that correspond to different files
|
||||
|
@ -47,13 +47,13 @@ namespace ErrorCodes
|
||||
{
|
||||
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
|
||||
extern const int INCORRECT_FILE_NAME;
|
||||
extern const int TIMEOUT_EXCEEDED;
|
||||
}
|
||||
|
||||
|
||||
class StripeLogSource final : public SourceWithProgress
|
||||
{
|
||||
public:
|
||||
|
||||
static Block getHeader(
|
||||
StorageStripeLog & storage,
|
||||
const StorageMetadataPtr & metadata_snapshot,
|
||||
@ -157,10 +157,11 @@ private:
|
||||
class StripeLogBlockOutputStream final : public IBlockOutputStream
|
||||
{
|
||||
public:
|
||||
explicit StripeLogBlockOutputStream(StorageStripeLog & storage_, const StorageMetadataPtr & metadata_snapshot_)
|
||||
explicit StripeLogBlockOutputStream(
|
||||
StorageStripeLog & storage_, const StorageMetadataPtr & metadata_snapshot_, std::unique_lock<std::shared_timed_mutex> && lock_)
|
||||
: storage(storage_)
|
||||
, metadata_snapshot(metadata_snapshot_)
|
||||
, lock(storage.rwlock)
|
||||
, lock(std::move(lock_))
|
||||
, data_out_file(storage.table_path + "data.bin")
|
||||
, data_out_compressed(storage.disk->writeFile(data_out_file, DBMS_DEFAULT_BUFFER_SIZE, WriteMode::Append))
|
||||
, data_out(std::make_unique<CompressedWriteBuffer>(
|
||||
@ -170,6 +171,8 @@ public:
|
||||
, index_out(std::make_unique<CompressedWriteBuffer>(*index_out_compressed))
|
||||
, block_out(*data_out, 0, metadata_snapshot->getSampleBlock(), false, index_out.get(), storage.disk->getFileSize(data_out_file))
|
||||
{
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
}
|
||||
|
||||
~StripeLogBlockOutputStream() override
|
||||
@ -223,7 +226,7 @@ public:
|
||||
private:
|
||||
StorageStripeLog & storage;
|
||||
StorageMetadataPtr metadata_snapshot;
|
||||
std::unique_lock<std::shared_mutex> lock;
|
||||
std::unique_lock<std::shared_timed_mutex> lock;
|
||||
|
||||
String data_out_file;
|
||||
std::unique_ptr<WriteBuffer> data_out_compressed;
|
||||
@ -286,8 +289,6 @@ void StorageStripeLog::rename(const String & new_path_to_table_data, const Stora
|
||||
{
|
||||
assert(table_path != new_path_to_table_data);
|
||||
{
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
disk->moveDirectory(table_path, new_path_to_table_data);
|
||||
|
||||
table_path = new_path_to_table_data;
|
||||
@ -297,6 +298,16 @@ void StorageStripeLog::rename(const String & new_path_to_table_data, const Stora
|
||||
}
|
||||
|
||||
|
||||
static std::chrono::seconds getLockTimeout(const Context & context)
|
||||
{
|
||||
const Settings & settings = context.getSettingsRef();
|
||||
Int64 lock_timeout = settings.lock_acquire_timeout.totalSeconds();
|
||||
if (settings.max_execution_time.totalSeconds() != 0 && settings.max_execution_time.totalSeconds() < lock_timeout)
|
||||
lock_timeout = settings.max_execution_time.totalSeconds();
|
||||
return std::chrono::seconds{lock_timeout};
|
||||
}
|
||||
|
||||
|
||||
Pipe StorageStripeLog::read(
|
||||
const Names & column_names,
|
||||
const StorageMetadataPtr & metadata_snapshot,
|
||||
@ -306,7 +317,9 @@ Pipe StorageStripeLog::read(
|
||||
const size_t /*max_block_size*/,
|
||||
unsigned num_streams)
|
||||
{
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
std::shared_lock lock(rwlock, getLockTimeout(context));
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
metadata_snapshot->check(column_names, getVirtuals(), getStorageID());
|
||||
|
||||
@ -345,24 +358,28 @@ Pipe StorageStripeLog::read(
|
||||
}
|
||||
|
||||
|
||||
BlockOutputStreamPtr StorageStripeLog::write(const ASTPtr & /*query*/, const StorageMetadataPtr & metadata_snapshot, const Context & /*context*/)
|
||||
BlockOutputStreamPtr StorageStripeLog::write(const ASTPtr & /*query*/, const StorageMetadataPtr & metadata_snapshot, const Context & context)
|
||||
{
|
||||
return std::make_shared<StripeLogBlockOutputStream>(*this, metadata_snapshot);
|
||||
std::unique_lock lock(rwlock, getLockTimeout(context));
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
return std::make_shared<StripeLogBlockOutputStream>(*this, metadata_snapshot, std::move(lock));
|
||||
}
|
||||
|
||||
|
||||
CheckResults StorageStripeLog::checkData(const ASTPtr & /* query */, const Context & /* context */)
|
||||
CheckResults StorageStripeLog::checkData(const ASTPtr & /* query */, const Context & context)
|
||||
{
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
std::shared_lock lock(rwlock, getLockTimeout(context));
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
return file_checker.check();
|
||||
}
|
||||
|
||||
void StorageStripeLog::truncate(const ASTPtr &, const StorageMetadataPtr &, const Context &, TableExclusiveLockHolder &)
|
||||
{
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
disk->clearDirectory(table_path);
|
||||
|
||||
file_checker = FileChecker{disk, table_path + "sizes.json"};
|
||||
}
|
||||
|
||||
|
@ -67,7 +67,7 @@ private:
|
||||
size_t max_compress_block_size;
|
||||
|
||||
FileChecker file_checker;
|
||||
mutable std::shared_mutex rwlock;
|
||||
mutable std::shared_timed_mutex rwlock;
|
||||
|
||||
Poco::Logger * log;
|
||||
};
|
||||
|
@ -13,6 +13,7 @@
|
||||
|
||||
#include <IO/ReadBufferFromFileBase.h>
|
||||
#include <IO/WriteBufferFromFileBase.h>
|
||||
#include <IO/LimitReadBuffer.h>
|
||||
#include <Compression/CompressionFactory.h>
|
||||
#include <Compression/CompressedReadBuffer.h>
|
||||
#include <Compression/CompressedWriteBuffer.h>
|
||||
@ -46,6 +47,7 @@ namespace DB
|
||||
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int TIMEOUT_EXCEEDED;
|
||||
extern const int DUPLICATE_COLUMN;
|
||||
extern const int INCORRECT_FILE_NAME;
|
||||
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
|
||||
@ -55,7 +57,6 @@ namespace ErrorCodes
|
||||
class TinyLogSource final : public SourceWithProgress
|
||||
{
|
||||
public:
|
||||
|
||||
static Block getHeader(const NamesAndTypesList & columns)
|
||||
{
|
||||
Block res;
|
||||
@ -66,10 +67,17 @@ public:
|
||||
return Nested::flatten(res);
|
||||
}
|
||||
|
||||
TinyLogSource(size_t block_size_, const NamesAndTypesList & columns_, StorageTinyLog & storage_, size_t max_read_buffer_size_)
|
||||
TinyLogSource(
|
||||
size_t block_size_,
|
||||
const NamesAndTypesList & columns_,
|
||||
StorageTinyLog & storage_,
|
||||
size_t max_read_buffer_size_,
|
||||
FileChecker::Map file_sizes_)
|
||||
: SourceWithProgress(getHeader(columns_))
|
||||
, block_size(block_size_), columns(columns_), storage(storage_), lock(storage_.rwlock)
|
||||
, max_read_buffer_size(max_read_buffer_size_) {}
|
||||
, block_size(block_size_), columns(columns_), storage(storage_)
|
||||
, max_read_buffer_size(max_read_buffer_size_), file_sizes(std::move(file_sizes_))
|
||||
{
|
||||
}
|
||||
|
||||
String getName() const override { return "TinyLog"; }
|
||||
|
||||
@ -80,19 +88,21 @@ private:
|
||||
size_t block_size;
|
||||
NamesAndTypesList columns;
|
||||
StorageTinyLog & storage;
|
||||
std::shared_lock<std::shared_mutex> lock;
|
||||
bool is_finished = false;
|
||||
size_t max_read_buffer_size;
|
||||
FileChecker::Map file_sizes;
|
||||
|
||||
struct Stream
|
||||
{
|
||||
Stream(const DiskPtr & disk, const String & data_path, size_t max_read_buffer_size_)
|
||||
Stream(const DiskPtr & disk, const String & data_path, size_t max_read_buffer_size_, size_t file_size)
|
||||
: plain(disk->readFile(data_path, std::min(max_read_buffer_size_, disk->getFileSize(data_path)))),
|
||||
limited(std::make_unique<LimitReadBuffer>(*plain, file_size, false)),
|
||||
compressed(*plain)
|
||||
{
|
||||
}
|
||||
|
||||
std::unique_ptr<ReadBuffer> plain;
|
||||
std::unique_ptr<ReadBuffer> limited;
|
||||
CompressedReadBuffer compressed;
|
||||
};
|
||||
|
||||
@ -110,9 +120,14 @@ private:
|
||||
class TinyLogBlockOutputStream final : public IBlockOutputStream
|
||||
{
|
||||
public:
|
||||
explicit TinyLogBlockOutputStream(StorageTinyLog & storage_, const StorageMetadataPtr & metadata_snapshot_)
|
||||
: storage(storage_), metadata_snapshot(metadata_snapshot_), lock(storage_.rwlock)
|
||||
explicit TinyLogBlockOutputStream(
|
||||
StorageTinyLog & storage_,
|
||||
const StorageMetadataPtr & metadata_snapshot_,
|
||||
std::unique_lock<std::shared_timed_mutex> && lock_)
|
||||
: storage(storage_), metadata_snapshot(metadata_snapshot_), lock(std::move(lock_))
|
||||
{
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
}
|
||||
|
||||
~TinyLogBlockOutputStream() override
|
||||
@ -140,7 +155,7 @@ public:
|
||||
private:
|
||||
StorageTinyLog & storage;
|
||||
StorageMetadataPtr metadata_snapshot;
|
||||
std::unique_lock<std::shared_mutex> lock;
|
||||
std::unique_lock<std::shared_timed_mutex> lock;
|
||||
bool done = false;
|
||||
|
||||
struct Stream
|
||||
@ -231,13 +246,17 @@ void TinyLogSource::readData(const String & name, const IDataType & type, IColum
|
||||
String stream_name = IDataType::getFileNameForStream(name, path);
|
||||
|
||||
if (!streams.count(stream_name))
|
||||
streams[stream_name] = std::make_unique<Stream>(storage.disk, storage.files[stream_name].data_file_path, max_read_buffer_size);
|
||||
{
|
||||
String file_path = storage.files[stream_name].data_file_path;
|
||||
streams[stream_name] = std::make_unique<Stream>(
|
||||
storage.disk, file_path, max_read_buffer_size, file_sizes[fileName(file_path)]);
|
||||
}
|
||||
|
||||
return &streams[stream_name]->compressed;
|
||||
};
|
||||
|
||||
if (deserialize_states.count(name) == 0)
|
||||
type.deserializeBinaryBulkStatePrefix(settings, deserialize_states[name]);
|
||||
type.deserializeBinaryBulkStatePrefix(settings, deserialize_states[name]);
|
||||
|
||||
type.deserializeBinaryBulkWithMultipleStreams(column, limit, settings, deserialize_states[name]);
|
||||
}
|
||||
@ -410,8 +429,6 @@ void StorageTinyLog::rename(const String & new_path_to_table_data, const Storage
|
||||
{
|
||||
assert(table_path != new_path_to_table_data);
|
||||
{
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
disk->moveDirectory(table_path, new_path_to_table_data);
|
||||
|
||||
table_path = new_path_to_table_data;
|
||||
@ -424,6 +441,16 @@ void StorageTinyLog::rename(const String & new_path_to_table_data, const Storage
|
||||
}
|
||||
|
||||
|
||||
static std::chrono::seconds getLockTimeout(const Context & context)
|
||||
{
|
||||
const Settings & settings = context.getSettingsRef();
|
||||
Int64 lock_timeout = settings.lock_acquire_timeout.totalSeconds();
|
||||
if (settings.max_execution_time.totalSeconds() != 0 && settings.max_execution_time.totalSeconds() < lock_timeout)
|
||||
lock_timeout = settings.max_execution_time.totalSeconds();
|
||||
return std::chrono::seconds{lock_timeout};
|
||||
}
|
||||
|
||||
|
||||
Pipe StorageTinyLog::read(
|
||||
const Names & column_names,
|
||||
const StorageMetadataPtr & metadata_snapshot,
|
||||
@ -437,28 +464,40 @@ Pipe StorageTinyLog::read(
|
||||
|
||||
// When reading, we lock the entire storage, because we only have one file
|
||||
// per column and can't modify it concurrently.
|
||||
const Settings & settings = context.getSettingsRef();
|
||||
|
||||
std::shared_lock lock{rwlock, getLockTimeout(context)};
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
/// No need to hold lock while reading because we read fixed range of data that does not change while appending more data.
|
||||
return Pipe(std::make_shared<TinyLogSource>(
|
||||
max_block_size, Nested::collect(metadata_snapshot->getColumns().getAllPhysical().addTypes(column_names)), *this, context.getSettingsRef().max_read_buffer_size));
|
||||
max_block_size,
|
||||
Nested::collect(metadata_snapshot->getColumns().getAllPhysical().addTypes(column_names)),
|
||||
*this,
|
||||
settings.max_read_buffer_size,
|
||||
file_checker.getFileSizes()));
|
||||
}
|
||||
|
||||
|
||||
BlockOutputStreamPtr StorageTinyLog::write(const ASTPtr & /*query*/, const StorageMetadataPtr & metadata_snapshot, const Context & /*context*/)
|
||||
BlockOutputStreamPtr StorageTinyLog::write(const ASTPtr & /*query*/, const StorageMetadataPtr & metadata_snapshot, const Context & context)
|
||||
{
|
||||
return std::make_shared<TinyLogBlockOutputStream>(*this, metadata_snapshot);
|
||||
return std::make_shared<TinyLogBlockOutputStream>(*this, metadata_snapshot, std::unique_lock{rwlock, getLockTimeout(context)});
|
||||
}
|
||||
|
||||
|
||||
CheckResults StorageTinyLog::checkData(const ASTPtr & /* query */, const Context & /* context */)
|
||||
CheckResults StorageTinyLog::checkData(const ASTPtr & /* query */, const Context & context)
|
||||
{
|
||||
std::shared_lock<std::shared_mutex> lock(rwlock);
|
||||
std::shared_lock lock(rwlock, getLockTimeout(context));
|
||||
if (!lock)
|
||||
throw Exception("Lock timeout exceeded", ErrorCodes::TIMEOUT_EXCEEDED);
|
||||
|
||||
return file_checker.check();
|
||||
}
|
||||
|
||||
void StorageTinyLog::truncate(
|
||||
const ASTPtr &, const StorageMetadataPtr & metadata_snapshot, const Context &, TableExclusiveLockHolder &)
|
||||
{
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
|
||||
disk->clearDirectory(table_path);
|
||||
|
||||
files.clear();
|
||||
@ -468,14 +507,6 @@ void StorageTinyLog::truncate(
|
||||
addFiles(column.name, *column.type);
|
||||
}
|
||||
|
||||
void StorageTinyLog::drop()
|
||||
{
|
||||
std::unique_lock<std::shared_mutex> lock(rwlock);
|
||||
if (disk->exists(table_path))
|
||||
disk->removeRecursive(table_path);
|
||||
files.clear();
|
||||
}
|
||||
|
||||
|
||||
void registerStorageTinyLog(StorageFactory & factory)
|
||||
{
|
||||
|
@ -43,8 +43,6 @@ public:
|
||||
|
||||
void truncate(const ASTPtr &, const StorageMetadataPtr & metadata_snapshot, const Context &, TableExclusiveLockHolder &) override;
|
||||
|
||||
void drop() override;
|
||||
|
||||
protected:
|
||||
StorageTinyLog(
|
||||
DiskPtr disk_,
|
||||
@ -70,7 +68,7 @@ private:
|
||||
Files files;
|
||||
|
||||
FileChecker file_checker;
|
||||
mutable std::shared_mutex rwlock;
|
||||
mutable std::shared_timed_mutex rwlock;
|
||||
|
||||
Poco::Logger * log;
|
||||
|
||||
|
@ -456,7 +456,7 @@ class BuildFlags(object):
|
||||
DEBUG = 'debug-build'
|
||||
UNBUNDLED = 'unbundled-build'
|
||||
RELEASE = 'release-build'
|
||||
DATABASE_ATOMIC = 'database-atomic'
|
||||
DATABASE_ORDINARY = 'database-ordinary'
|
||||
POLYMORPHIC_PARTS = 'polymorphic-parts'
|
||||
|
||||
|
||||
@ -501,8 +501,8 @@ def collect_build_flags(client):
|
||||
(stdout, stderr) = clickhouse_proc.communicate("SELECT value FROM system.settings WHERE name = 'default_database_engine'")
|
||||
|
||||
if clickhouse_proc.returncode == 0:
|
||||
if 'Atomic' in stdout:
|
||||
result.append(BuildFlags.DATABASE_ATOMIC)
|
||||
if 'Ordinary' in stdout:
|
||||
result.append(BuildFlags.DATABASE_ORDINARY)
|
||||
else:
|
||||
raise Exception("Cannot get inforamtion about build from server errorcode {}, stderr {}".format(clickhouse_proc.returncode, stderr))
|
||||
|
||||
|
8
tests/config/README.md
Normal file
8
tests/config/README.md
Normal file
@ -0,0 +1,8 @@
|
||||
# ClickHouse configs for test environment
|
||||
|
||||
## How to use
|
||||
CI use these configs in all checks installing them with `install.sh` script. If you want to run all tests from `tests/queries/0_stateless` and `test/queries/1_stateful` on your local machine you have to set up configs from this directory for your `clickhouse-server`. The most simple way is to install them using `install.sh` script. Other option is just copy files into your clickhouse config directory.
|
||||
|
||||
## How to add new config
|
||||
|
||||
Just place file `.xml` with new config into appropriate directory and add `ln` command into `install.sh` script. After that CI will use this config in all tests runs.
|
@ -1,7 +0,0 @@
|
||||
<yandex>
|
||||
<profiles>
|
||||
<default>
|
||||
<show_table_uuid_in_table_create_query_if_not_nil>0</show_table_uuid_in_table_create_query_if_not_nil>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
54
tests/config/install.sh
Executable file
54
tests/config/install.sh
Executable file
@ -0,0 +1,54 @@
|
||||
#!/bin/bash
|
||||
|
||||
# script allows to install configs for clickhouse server and clients required
|
||||
# for testing (stateless and stateful tests)
|
||||
|
||||
set -x -e
|
||||
|
||||
DEST_SERVER_PATH="${1:-/etc/clickhouse-server}"
|
||||
DEST_CLIENT_PATH="${2:-/etc/clickhouse-client}"
|
||||
SRC_PATH="$( cd "$(dirname "$0")" >/dev/null 2>&1 ; pwd -P )"
|
||||
|
||||
echo "Going to install test configs from $SRC_PATH into $DEST_SERVER_PATH"
|
||||
|
||||
mkdir -p $DEST_SERVER_PATH/config.d/
|
||||
mkdir -p $DEST_SERVER_PATH/users.d/
|
||||
mkdir -p $DEST_CLIENT_PATH
|
||||
|
||||
ln -s $SRC_PATH/config.d/zookeeper.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/listen.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/part_log.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/text_log.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/metric_log.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/custom_settings_prefixes.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/macros.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/disks.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/secure_ports.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/clusters.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/graphite.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/config.d/database_atomic.xml $DEST_SERVER_PATH/config.d/
|
||||
ln -s $SRC_PATH/users.d/log_queries.xml $DEST_SERVER_PATH/users.d/
|
||||
ln -s $SRC_PATH/users.d/readonly.xml $DEST_SERVER_PATH/users.d/
|
||||
ln -s $SRC_PATH/users.d/access_management.xml $DEST_SERVER_PATH/users.d/
|
||||
|
||||
ln -s $SRC_PATH/ints_dictionary.xml $DEST_SERVER_PATH/
|
||||
ln -s $SRC_PATH/strings_dictionary.xml $DEST_SERVER_PATH/
|
||||
ln -s $SRC_PATH/decimals_dictionary.xml $DEST_SERVER_PATH/
|
||||
ln -s $SRC_PATH/executable_dictionary.xml $DEST_SERVER_PATH/
|
||||
|
||||
ln -s $SRC_PATH/server.key $DEST_SERVER_PATH/
|
||||
ln -s $SRC_PATH/server.crt $DEST_SERVER_PATH/
|
||||
ln -s $SRC_PATH/dhparam.pem $DEST_SERVER_PATH/
|
||||
|
||||
# Retain any pre-existing config and allow ClickHouse to load it if required
|
||||
ln -s --backup=simple --suffix=_original.xml \
|
||||
$SRC_PATH/config.d/query_masking_rules.xml $DEST_SERVER_PATH/config.d/
|
||||
|
||||
if [[ -n "$USE_POLYMORPHIC_PARTS" ]] && [[ "$USE_POLYMORPHIC_PARTS" -eq 1 ]]; then
|
||||
ln -s $SRC_PATH/config.d/polymorphic_parts.xml $DEST_SERVER_PATH/config.d/
|
||||
fi
|
||||
if [[ -n "$USE_DATABASE_ORDINARY" ]] && [[ "$USE_DATABASE_ORDINARY" -eq 1 ]]; then
|
||||
ln -s $SRC_PATH/users.d/database_ordinary.xml $DEST_SERVER_PATH/users.d/
|
||||
fi
|
||||
|
||||
ln -sf $SRC_PATH/client_config.xml $DEST_CLIENT_PATH/config.xml
|
@ -45,6 +45,7 @@ def _create_env_file(path, variables, fname=DEFAULT_ENV_NAME):
|
||||
f.write("=".join([var, value]) + "\n")
|
||||
return full_path
|
||||
|
||||
|
||||
def subprocess_check_call(args):
|
||||
# Uncomment for debugging
|
||||
# print('run:', ' ' . join(args))
|
||||
@ -124,7 +125,6 @@ class ClickHouseCluster:
|
||||
self.base_zookeeper_cmd = None
|
||||
self.base_mysql_cmd = []
|
||||
self.base_kafka_cmd = []
|
||||
self.base_kerberized_kafka_cmd = []
|
||||
self.base_rabbitmq_cmd = []
|
||||
self.base_cassandra_cmd = []
|
||||
self.pre_zookeeper_commands = []
|
||||
@ -133,7 +133,6 @@ class ClickHouseCluster:
|
||||
self.with_mysql = False
|
||||
self.with_postgres = False
|
||||
self.with_kafka = False
|
||||
self.with_kerberized_kafka = False
|
||||
self.with_rabbitmq = False
|
||||
self.with_odbc_drivers = False
|
||||
self.with_hdfs = False
|
||||
@ -170,7 +169,7 @@ class ClickHouseCluster:
|
||||
|
||||
def add_instance(self, name, base_config_dir=None, main_configs=None, user_configs=None, dictionaries=None,
|
||||
macros=None,
|
||||
with_zookeeper=False, with_mysql=False, with_kafka=False, with_kerberized_kafka=False, with_rabbitmq=False,
|
||||
with_zookeeper=False, with_mysql=False, with_kafka=False, with_rabbitmq=False,
|
||||
clickhouse_path_dir=None,
|
||||
with_odbc_drivers=False, with_postgres=False, with_hdfs=False, with_mongo=False,
|
||||
with_redis=False, with_minio=False, with_cassandra=False,
|
||||
@ -208,7 +207,6 @@ class ClickHouseCluster:
|
||||
zookeeper_config_path=self.zookeeper_config_path,
|
||||
with_mysql=with_mysql,
|
||||
with_kafka=with_kafka,
|
||||
with_kerberized_kafka=with_kerberized_kafka,
|
||||
with_rabbitmq=with_rabbitmq,
|
||||
with_mongo=with_mongo,
|
||||
with_redis=with_redis,
|
||||
@ -292,13 +290,6 @@ class ClickHouseCluster:
|
||||
p.join(docker_compose_yml_dir, 'docker_compose_kafka.yml')]
|
||||
cmds.append(self.base_kafka_cmd)
|
||||
|
||||
if with_kerberized_kafka and not self.with_kerberized_kafka:
|
||||
self.with_kerberized_kafka = True
|
||||
self.base_cmd.extend(['--file', p.join(docker_compose_yml_dir, 'docker_compose_kerberized_kafka.yml')])
|
||||
self.base_kerberized_kafka_cmd = ['docker-compose', '--project-directory', self.base_dir, '--project-name',
|
||||
self.project_name, '--file', p.join(docker_compose_yml_dir, 'docker_compose_kerberized_kafka.yml')]
|
||||
cmds.append(self.base_kerberized_kafka_cmd)
|
||||
|
||||
if with_rabbitmq and not self.with_rabbitmq:
|
||||
self.with_rabbitmq = True
|
||||
self.base_cmd.extend(['--file', p.join(docker_compose_yml_dir, 'docker_compose_rabbitmq.yml')])
|
||||
@ -617,11 +608,6 @@ class ClickHouseCluster:
|
||||
self.kafka_docker_id = self.get_instance_docker_id('kafka1')
|
||||
self.wait_schema_registry_to_start(120)
|
||||
|
||||
if self.with_kerberized_kafka and self.base_kerberized_kafka_cmd:
|
||||
env = os.environ.copy()
|
||||
env['KERBERIZED_KAFKA_DIR'] = instance.path + '/'
|
||||
subprocess.check_call(self.base_kerberized_kafka_cmd + common_opts + ['--renew-anon-volumes'], env=env)
|
||||
self.kerberized_kafka_docker_id = self.get_instance_docker_id('kerberized_kafka1')
|
||||
if self.with_rabbitmq and self.base_rabbitmq_cmd:
|
||||
subprocess_check_call(self.base_rabbitmq_cmd + common_opts + ['--renew-anon-volumes'])
|
||||
self.rabbitmq_docker_id = self.get_instance_docker_id('rabbitmq1')
|
||||
@ -802,12 +788,9 @@ services:
|
||||
- {instance_config_dir}:/etc/clickhouse-server/
|
||||
- {db_dir}:/var/lib/clickhouse/
|
||||
- {logs_dir}:/var/log/clickhouse-server/
|
||||
- /etc/passwd:/etc/passwd:ro
|
||||
{binary_volume}
|
||||
{odbc_bridge_volume}
|
||||
{odbc_ini_path}
|
||||
{keytab_path}
|
||||
{krb5_conf}
|
||||
entrypoint: {entrypoint_cmd}
|
||||
tmpfs: {tmpfs}
|
||||
cap_add:
|
||||
@ -837,7 +820,7 @@ class ClickHouseInstance:
|
||||
def __init__(
|
||||
self, cluster, base_path, name, base_config_dir, custom_main_configs, custom_user_configs,
|
||||
custom_dictionaries,
|
||||
macros, with_zookeeper, zookeeper_config_path, with_mysql, with_kafka, with_kerberized_kafka, with_rabbitmq, with_mongo,
|
||||
macros, with_zookeeper, zookeeper_config_path, with_mysql, with_kafka, with_rabbitmq, with_mongo,
|
||||
with_redis, with_minio,
|
||||
with_cassandra, server_bin_path, odbc_bridge_bin_path, clickhouse_path_dir, with_odbc_drivers,
|
||||
hostname=None, env_variables=None,
|
||||
@ -856,7 +839,6 @@ class ClickHouseInstance:
|
||||
self.custom_user_config_paths = [p.abspath(p.join(base_path, c)) for c in custom_user_configs]
|
||||
self.custom_dictionaries_paths = [p.abspath(p.join(base_path, c)) for c in custom_dictionaries]
|
||||
self.clickhouse_path_dir = p.abspath(p.join(base_path, clickhouse_path_dir)) if clickhouse_path_dir else None
|
||||
self.kerberos_secrets_dir = p.abspath(p.join(base_path, 'secrets'))
|
||||
self.macros = macros if macros is not None else {}
|
||||
self.with_zookeeper = with_zookeeper
|
||||
self.zookeeper_config_path = zookeeper_config_path
|
||||
@ -866,7 +848,6 @@ class ClickHouseInstance:
|
||||
|
||||
self.with_mysql = with_mysql
|
||||
self.with_kafka = with_kafka
|
||||
self.with_kerberized_kafka = with_kerberized_kafka
|
||||
self.with_rabbitmq = with_rabbitmq
|
||||
self.with_mongo = with_mongo
|
||||
self.with_redis = with_redis
|
||||
@ -882,13 +863,6 @@ class ClickHouseInstance:
|
||||
else:
|
||||
self.odbc_ini_path = ""
|
||||
|
||||
if with_kerberized_kafka:
|
||||
self.keytab_path = '- ' + os.path.dirname(self.docker_compose_path) + "/secrets:/tmp/keytab"
|
||||
self.krb5_conf = '- ' + os.path.dirname(self.docker_compose_path) + "/secrets/krb.conf:/etc/krb5.conf:ro"
|
||||
else:
|
||||
self.keytab_path = ""
|
||||
self.krb5_conf = ""
|
||||
|
||||
self.docker_client = None
|
||||
self.ip_address = None
|
||||
self.client = None
|
||||
@ -1218,9 +1192,6 @@ class ClickHouseInstance:
|
||||
if self.with_zookeeper:
|
||||
shutil.copy(self.zookeeper_config_path, conf_d_dir)
|
||||
|
||||
if self.with_kerberized_kafka:
|
||||
shutil.copytree(self.kerberos_secrets_dir, p.abspath(p.join(self.path, 'secrets')))
|
||||
|
||||
# Copy config.d configs
|
||||
print "Copy custom test config files {} to {}".format(self.custom_main_config_paths, self.config_d_dir)
|
||||
for path in self.custom_main_config_paths:
|
||||
@ -1256,9 +1227,6 @@ class ClickHouseInstance:
|
||||
depends_on.append("kafka1")
|
||||
depends_on.append("schema-registry")
|
||||
|
||||
if self.with_kerberized_kafka:
|
||||
depends_on.append("kerberized_kafka1")
|
||||
|
||||
if self.with_rabbitmq:
|
||||
depends_on.append("rabbitmq1")
|
||||
|
||||
@ -1322,8 +1290,6 @@ class ClickHouseInstance:
|
||||
user=os.getuid(),
|
||||
env_file=env_file,
|
||||
odbc_ini_path=odbc_ini_path,
|
||||
keytab_path=self.keytab_path,
|
||||
krb5_conf=self.krb5_conf,
|
||||
entrypoint_cmd=entrypoint_cmd,
|
||||
networks=networks,
|
||||
app_net=app_net,
|
||||
|
@ -60,3 +60,19 @@ def assert_eq_with_retry(instance, query, expectation, retry_count=20, sleep_tim
|
||||
if expectation_tsv != val:
|
||||
raise AssertionError("'{}' != '{}'\n{}".format(expectation_tsv, val, '\n'.join(
|
||||
expectation_tsv.diff(val, n1="expectation", n2="query"))))
|
||||
|
||||
def assert_logs_contain(instance, substring):
|
||||
if not instance.contains_in_log(substring):
|
||||
raise AssertionError("'{}' not found in logs".format(substring))
|
||||
|
||||
def assert_logs_contain_with_retry(instance, substring, retry_count=20, sleep_time=0.5):
|
||||
for i in xrange(retry_count):
|
||||
try:
|
||||
if instance.contains_in_log(substring):
|
||||
break
|
||||
time.sleep(sleep_time)
|
||||
except Exception as ex:
|
||||
print "contains_in_log_with_retry retry {} exception {}".format(i + 1, ex)
|
||||
time.sleep(sleep_time)
|
||||
else:
|
||||
raise AssertionError("'{}' not found in logs".format(substring))
|
||||
|
@ -156,8 +156,6 @@ if __name__ == "__main__":
|
||||
env_tags += "-e {}={} ".format("DOCKER_POSTGRESQL_JAVA_CLIENT_TAG", tag)
|
||||
elif image == "yandex/clickhouse-integration-test":
|
||||
env_tags += "-e {}={}".format("DOCKER_BASE_TAG", tag)
|
||||
elif image == "yandex/clickhouse-kerberos-kdc":
|
||||
env_tags += "-e {}={}".format("DOCKER_KERBEROS_KDC_TAG", tag)
|
||||
else:
|
||||
logging.info("Unknown image {}".format(image))
|
||||
|
||||
|
@ -1,3 +0,0 @@
|
||||
<yandex>
|
||||
<text_log/>
|
||||
</yandex>
|
@ -6,13 +6,5 @@
|
||||
<custom_c>Float64_-43.25e-1</custom_c>
|
||||
<custom_d>'some text'</custom_d>
|
||||
</default>
|
||||
|
||||
<profile_with_unknown_setting>
|
||||
<x>1</x>
|
||||
</profile_with_unknown_setting>
|
||||
|
||||
<profile_illformed_setting>
|
||||
<custom_f>1</custom_f>
|
||||
</profile_illformed_setting>
|
||||
</profiles>
|
||||
</yandex>
|
@ -0,0 +1,7 @@
|
||||
<yandex>
|
||||
<profiles>
|
||||
<default>
|
||||
<custom_f>1</custom_f>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
@ -1,9 +1,10 @@
|
||||
import pytest
|
||||
import os
|
||||
from helpers.cluster import ClickHouseCluster
|
||||
|
||||
SCRIPT_DIR = os.path.dirname(os.path.realpath(__file__))
|
||||
cluster = ClickHouseCluster(__file__)
|
||||
node = cluster.add_instance('node', main_configs=["configs/config.d/text_log.xml"],
|
||||
user_configs=["configs/users.d/custom_settings.xml"])
|
||||
node = cluster.add_instance('node')
|
||||
|
||||
|
||||
@pytest.fixture(scope="module", autouse=True)
|
||||
@ -16,28 +17,17 @@ def started_cluster():
|
||||
cluster.shutdown()
|
||||
|
||||
|
||||
def test():
|
||||
def test_custom_settings():
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/custom_settings.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
node.query("SYSTEM RELOAD CONFIG")
|
||||
|
||||
assert node.query("SELECT getSetting('custom_a')") == "-5\n"
|
||||
assert node.query("SELECT getSetting('custom_b')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('custom_c')") == "-4.325\n"
|
||||
assert node.query("SELECT getSetting('custom_d')") == "some text\n"
|
||||
|
||||
assert "custom_a = -5, custom_b = 10000000000, custom_c = -4.325, custom_d = \\'some text\\'" \
|
||||
in node.query("SHOW CREATE SETTINGS PROFILE default")
|
||||
|
||||
assert "no settings profile" in node.query_and_get_error(
|
||||
"SHOW CREATE SETTINGS PROFILE profile_with_unknown_setting")
|
||||
assert "no settings profile" in node.query_and_get_error("SHOW CREATE SETTINGS PROFILE profile_illformed_setting")
|
||||
|
||||
|
||||
def test_invalid_settings():
|
||||
node.query("SYSTEM RELOAD CONFIG")
|
||||
node.query("SYSTEM FLUSH LOGS")
|
||||
|
||||
assert node.query("SELECT COUNT() FROM system.text_log WHERE"
|
||||
" message LIKE '%Could not parse profile `profile_illformed_setting`%'"
|
||||
" AND message LIKE '%Couldn\\'t restore Field from dump%'") == "1\n"
|
||||
|
||||
assert node.query("SELECT COUNT() FROM system.text_log WHERE"
|
||||
" message LIKE '%Could not parse profile `profile_with_unknown_setting`%'"
|
||||
" AND message LIKE '%Setting x is neither a builtin setting nor started with the prefix \\'custom_\\'%'") == "1\n"
|
||||
def test_illformed_setting():
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/illformed_setting.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
error_message = "Couldn't restore Field from dump: 1"
|
||||
assert error_message in node.query_and_get_error("SYSTEM RELOAD CONFIG")
|
||||
|
@ -0,0 +1,9 @@
|
||||
<?xml version="1.0"?>
|
||||
<yandex>
|
||||
<profiles replace="replace">
|
||||
<default>
|
||||
<max_memory_usage>20000000000</max_memory_usage>
|
||||
<load_balancing>nearest_hostname</load_balancing>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
@ -0,0 +1,9 @@
|
||||
<?xml version="1.0"?>
|
||||
<yandex>
|
||||
<profiles replace="replace">
|
||||
<default>
|
||||
<max_memory_usage>10000000000</max_memory_usage>
|
||||
<load_balancing>first_or_random</load_balancing>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
@ -0,0 +1,9 @@
|
||||
<?xml version="1.0"?>
|
||||
<yandex>
|
||||
<profiles replace="replace">
|
||||
<default>
|
||||
<max_memory_usage>20000000000</max_memory_usage>
|
||||
<load_balancing>a</load_balancing>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
@ -0,0 +1,9 @@
|
||||
<?xml version="1.0"?>
|
||||
<yandex>
|
||||
<profiles replace="replace">
|
||||
<default>
|
||||
<max_memory_usage>a</max_memory_usage>
|
||||
<load_balancing>nearest_hostname</load_balancing>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
@ -0,0 +1,8 @@
|
||||
<?xml version="1.0"?>
|
||||
<yandex>
|
||||
<profiles replace="replace">
|
||||
<default>
|
||||
<xyz>8</xyz>
|
||||
</default>
|
||||
</profiles>
|
||||
</yandex>
|
@ -0,0 +1,90 @@
|
||||
import pytest
|
||||
import os
|
||||
import time
|
||||
from helpers.cluster import ClickHouseCluster
|
||||
from helpers.test_tools import assert_eq_with_retry, assert_logs_contain_with_retry
|
||||
|
||||
SCRIPT_DIR = os.path.dirname(os.path.realpath(__file__))
|
||||
cluster = ClickHouseCluster(__file__)
|
||||
node = cluster.add_instance('node', user_configs=["configs/normal_settings.xml"])
|
||||
|
||||
@pytest.fixture(scope="module", autouse=True)
|
||||
def started_cluster():
|
||||
try:
|
||||
cluster.start()
|
||||
yield cluster
|
||||
finally:
|
||||
cluster.shutdown()
|
||||
|
||||
|
||||
@pytest.fixture(autouse=True)
|
||||
def reset_to_normal_settings_after_test():
|
||||
try:
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/normal_settings.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
node.query("SYSTEM RELOAD CONFIG")
|
||||
yield
|
||||
finally:
|
||||
pass
|
||||
|
||||
|
||||
def test_force_reload():
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "first_or_random\n"
|
||||
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/changed_settings.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
node.query("SYSTEM RELOAD CONFIG")
|
||||
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "20000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "nearest_hostname\n"
|
||||
|
||||
|
||||
def test_reload_on_timeout():
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "first_or_random\n"
|
||||
|
||||
time.sleep(1) # The modification time of the 'z.xml' file should be different,
|
||||
# because config files are reload by timer only when the modification time is changed.
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/changed_settings.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
|
||||
assert_eq_with_retry(node, "SELECT getSetting('max_memory_usage')", "20000000000")
|
||||
assert_eq_with_retry(node, "SELECT getSetting('load_balancing')", "nearest_hostname")
|
||||
|
||||
|
||||
def test_unknown_setting_force_reload():
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/unknown_setting.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
|
||||
error_message = "Setting xyz is neither a builtin setting nor started with the prefix 'custom_' registered for user-defined settings"
|
||||
assert error_message in node.query_and_get_error("SYSTEM RELOAD CONFIG")
|
||||
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "first_or_random\n"
|
||||
|
||||
|
||||
def test_unknown_setting_reload_on_timeout():
|
||||
time.sleep(1) # The modification time of the 'z.xml' file should be different,
|
||||
# because config files are reload by timer only when the modification time is changed.
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/unknown_setting.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
|
||||
error_message = "Setting xyz is neither a builtin setting nor started with the prefix 'custom_' registered for user-defined settings"
|
||||
assert_logs_contain_with_retry(node, error_message)
|
||||
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "first_or_random\n"
|
||||
|
||||
|
||||
def test_unexpected_setting_int():
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/unexpected_setting_int.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
error_message = "Cannot parse"
|
||||
assert error_message in node.query_and_get_error("SYSTEM RELOAD CONFIG")
|
||||
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "first_or_random\n"
|
||||
|
||||
|
||||
def test_unexpected_setting_enum():
|
||||
node.copy_file_to_container(os.path.join(SCRIPT_DIR, "configs/unexpected_setting_int.xml"), '/etc/clickhouse-server/users.d/z.xml')
|
||||
error_message = "Cannot parse"
|
||||
assert error_message in node.query_and_get_error("SYSTEM RELOAD CONFIG")
|
||||
|
||||
assert node.query("SELECT getSetting('max_memory_usage')") == "10000000000\n"
|
||||
assert node.query("SELECT getSetting('load_balancing')") == "first_or_random\n"
|
@ -1,26 +0,0 @@
|
||||
<yandex>
|
||||
<kafka>
|
||||
<auto_offset_reset>earliest</auto_offset_reset>
|
||||
<!-- Debugging of possible issues, like:
|
||||
- https://github.com/edenhill/librdkafka/issues/2077
|
||||
- https://github.com/edenhill/librdkafka/issues/1778
|
||||
- #5615
|
||||
|
||||
XXX: for now this messages will appears in stderr.
|
||||
-->
|
||||
<security_protocol>SASL_PLAINTEXT</security_protocol>
|
||||
<sasl_mechanism>GSSAPI</sasl_mechanism>
|
||||
<sasl_kerberos_service_name>kafka</sasl_kerberos_service_name>
|
||||
<sasl_kerberos_keytab>/tmp/keytab/clickhouse.keytab</sasl_kerberos_keytab>
|
||||
<sasl_kerberos_principal>kafkauser/instance@TEST.CLICKHOUSE.TECH</sasl_kerberos_principal>
|
||||
<debug>security</debug>
|
||||
<api_version_request>false</api_version_request>
|
||||
</kafka>
|
||||
|
||||
<kafka_consumer_hang>
|
||||
<!-- default: 3000 -->
|
||||
<heartbeat_interval_ms>300</heartbeat_interval_ms>
|
||||
<!-- default: 10000 -->
|
||||
<session_timeout_ms>6000</session_timeout_ms>
|
||||
</kafka_consumer_hang>
|
||||
</yandex>
|
@ -1,11 +0,0 @@
|
||||
<yandex>
|
||||
<logger>
|
||||
<level>trace</level>
|
||||
<log>/var/log/clickhouse-server/log.log</log>
|
||||
<errorlog>/var/log/clickhouse-server/log.err.log</errorlog>
|
||||
<size>1000M</size>
|
||||
<count>10</count>
|
||||
<stderr>/var/log/clickhouse-server/stderr.log</stderr>
|
||||
<stdout>/var/log/clickhouse-server/stdout.log</stdout>
|
||||
</logger>
|
||||
</yandex>
|
@ -1,132 +0,0 @@
|
||||
#!/bin/bash
|
||||
|
||||
|
||||
set -x # trace
|
||||
|
||||
: "${REALM:=TEST.CLICKHOUSE.TECH}"
|
||||
: "${DOMAIN_REALM:=test.clickhouse.tech}"
|
||||
: "${KERB_MASTER_KEY:=masterkey}"
|
||||
: "${KERB_ADMIN_USER:=admin}"
|
||||
: "${KERB_ADMIN_PASS:=admin}"
|
||||
|
||||
create_config() {
|
||||
: "${KDC_ADDRESS:=$(hostname -f)}"
|
||||
|
||||
cat>/etc/krb5.conf<<EOF
|
||||
[logging]
|
||||
default = FILE:/var/log/kerberos/krb5libs.log
|
||||
kdc = FILE:/var/log/kerberos/krb5kdc.log
|
||||
admin_server = FILE:/var/log/kerberos/kadmind.log
|
||||
|
||||
[libdefaults]
|
||||
default_realm = $REALM
|
||||
dns_lookup_realm = false
|
||||
dns_lookup_kdc = false
|
||||
ticket_lifetime = 15s
|
||||
renew_lifetime = 15s
|
||||
forwardable = true
|
||||
# WARNING: We use weaker key types to simplify testing as stronger key types
|
||||
# require the enhanced security JCE policy file to be installed. You should
|
||||
# NOT run with this configuration in production or any real environment. You
|
||||
# have been warned.
|
||||
default_tkt_enctypes = des-cbc-md5 des-cbc-crc des3-cbc-sha1
|
||||
default_tgs_enctypes = des-cbc-md5 des-cbc-crc des3-cbc-sha1
|
||||
permitted_enctypes = des-cbc-md5 des-cbc-crc des3-cbc-sha1
|
||||
|
||||
[realms]
|
||||
$REALM = {
|
||||
kdc = $KDC_ADDRESS
|
||||
admin_server = $KDC_ADDRESS
|
||||
}
|
||||
|
||||
[domain_realm]
|
||||
.$DOMAIN_REALM = $REALM
|
||||
$DOMAIN_REALM = $REALM
|
||||
EOF
|
||||
|
||||
cat>/var/kerberos/krb5kdc/kdc.conf<<EOF
|
||||
[kdcdefaults]
|
||||
kdc_ports = 88
|
||||
kdc_tcp_ports = 88
|
||||
|
||||
[realms]
|
||||
$REALM = {
|
||||
acl_file = /var/kerberos/krb5kdc/kadm5.acl
|
||||
dict_file = /usr/share/dict/words
|
||||
admin_keytab = /var/kerberos/krb5kdc/kadm5.keytab
|
||||
# WARNING: We use weaker key types to simplify testing as stronger key types
|
||||
# require the enhanced security JCE policy file to be installed. You should
|
||||
# NOT run with this configuration in production or any real environment. You
|
||||
# have been warned.
|
||||
master_key_type = des3-hmac-sha1
|
||||
supported_enctypes = arcfour-hmac:normal des3-hmac-sha1:normal des-cbc-crc:normal des:normal des:v4 des:norealm des:onlyrealm des:afs3
|
||||
default_principal_flags = +preauth
|
||||
}
|
||||
EOF
|
||||
}
|
||||
|
||||
create_db() {
|
||||
/usr/sbin/kdb5_util -P $KERB_MASTER_KEY -r $REALM create -s
|
||||
}
|
||||
|
||||
start_kdc() {
|
||||
mkdir -p /var/log/kerberos
|
||||
|
||||
/etc/rc.d/init.d/krb5kdc start
|
||||
/etc/rc.d/init.d/kadmin start
|
||||
|
||||
chkconfig krb5kdc on
|
||||
chkconfig kadmin on
|
||||
}
|
||||
|
||||
restart_kdc() {
|
||||
/etc/rc.d/init.d/krb5kdc restart
|
||||
/etc/rc.d/init.d/kadmin restart
|
||||
}
|
||||
|
||||
create_admin_user() {
|
||||
kadmin.local -q "addprinc -pw $KERB_ADMIN_PASS $KERB_ADMIN_USER/admin"
|
||||
echo "*/admin@$REALM *" > /var/kerberos/krb5kdc/kadm5.acl
|
||||
}
|
||||
|
||||
create_keytabs() {
|
||||
|
||||
kadmin.local -q "addprinc -randkey zookeeper/kafka_kerberized_zookeeper@${REALM}"
|
||||
kadmin.local -q "ktadd -norandkey -k /tmp/keytab/kafka_kerberized_zookeeper.keytab zookeeper/kafka_kerberized_zookeeper@${REALM}"
|
||||
|
||||
kadmin.local -q "addprinc -randkey kafka/kerberized_kafka1@${REALM}"
|
||||
kadmin.local -q "ktadd -norandkey -k /tmp/keytab/kerberized_kafka.keytab kafka/kerberized_kafka1@${REALM}"
|
||||
|
||||
kadmin.local -q "addprinc -randkey zkclient@${REALM}"
|
||||
kadmin.local -q "ktadd -norandkey -k /tmp/keytab/zkclient.keytab zkclient@${REALM}"
|
||||
|
||||
|
||||
kadmin.local -q "addprinc -randkey kafkauser/instance@${REALM}"
|
||||
kadmin.local -q "ktadd -norandkey -k /tmp/keytab/clickhouse.keytab kafkauser/instance@${REALM}"
|
||||
|
||||
chmod g+r /tmp/keytab/clickhouse.keytab
|
||||
|
||||
}
|
||||
|
||||
main() {
|
||||
|
||||
if [ ! -f /kerberos_initialized ]; then
|
||||
create_config
|
||||
create_db
|
||||
create_admin_user
|
||||
start_kdc
|
||||
|
||||
touch /kerberos_initialized
|
||||
fi
|
||||
|
||||
if [ ! -f /var/kerberos/krb5kdc/principal ]; then
|
||||
while true; do sleep 1000; done
|
||||
else
|
||||
start_kdc
|
||||
create_keytabs
|
||||
tail -F /var/log/kerberos/krb5kdc.log
|
||||
fi
|
||||
|
||||
}
|
||||
|
||||
[[ "$0" == "${BASH_SOURCE[0]}" ]] && main "$@"
|
@ -1,14 +0,0 @@
|
||||
KafkaServer {
|
||||
com.sun.security.auth.module.Krb5LoginModule required
|
||||
useKeyTab=true
|
||||
storeKey=true
|
||||
keyTab="/etc/kafka/secrets/kerberized_kafka.keytab"
|
||||
principal="kafka/kerberized_kafka1@TEST.CLICKHOUSE.TECH";
|
||||
};
|
||||
Client {
|
||||
com.sun.security.auth.module.Krb5LoginModule required
|
||||
useKeyTab=true
|
||||
storeKey=true
|
||||
keyTab="/etc/kafka/secrets/zkclient.keytab"
|
||||
principal="zkclient@TEST.CLICKHOUSE.TECH";
|
||||
};
|
@ -1,22 +0,0 @@
|
||||
[logging]
|
||||
default = FILE:/var/log/kerberos/krb5libs.log
|
||||
kdc = FILE:/var/log/kerberos/krb5kdc.log
|
||||
admin_server = FILE:/var/log/kerberos/kadmind.log
|
||||
|
||||
[libdefaults]
|
||||
default_realm = TEST.CLICKHOUSE.TECH
|
||||
dns_lookup_realm = false
|
||||
dns_lookup_kdc = false
|
||||
ticket_lifetime = 15s
|
||||
renew_lifetime = 15s
|
||||
forwardable = true
|
||||
|
||||
[realms]
|
||||
TEST.CLICKHOUSE.TECH = {
|
||||
kdc = kafka_kerberos
|
||||
admin_server = kafka_kerberos
|
||||
}
|
||||
|
||||
[domain_realm]
|
||||
.TEST.CLICKHOUSE.TECH = TEST.CLICKHOUSE.TECH
|
||||
TEST.CLICKHOUSE.TECH = TEST.CLICKHOUSE.TECH
|
@ -1,14 +0,0 @@
|
||||
Server {
|
||||
com.sun.security.auth.module.Krb5LoginModule required
|
||||
useKeyTab=true
|
||||
storeKey=true
|
||||
keyTab="/etc/kafka/secrets/kafka_kerberized_zookeeper.keytab"
|
||||
principal="zookeeper/kafka_kerberized_zookeeper@TEST.CLICKHOUSE.TECH";
|
||||
};
|
||||
Client {
|
||||
com.sun.security.auth.module.Krb5LoginModule required
|
||||
useKeyTab=true
|
||||
storeKey=true
|
||||
keyTab="/etc/kafka/secrets/zkclient.keytab"
|
||||
principal="zkclient@TEST.CLICKHOUSE.TECH";
|
||||
};
|
@ -1,146 +0,0 @@
|
||||
import os.path as p
|
||||
import random
|
||||
import threading
|
||||
import time
|
||||
import pytest
|
||||
|
||||
from helpers.cluster import ClickHouseCluster
|
||||
from helpers.test_tools import TSV
|
||||
from helpers.client import QueryRuntimeException
|
||||
from helpers.network import PartitionManager
|
||||
|
||||
import json
|
||||
import subprocess
|
||||
import kafka.errors
|
||||
from kafka import KafkaAdminClient, KafkaProducer, KafkaConsumer, BrokerConnection
|
||||
from kafka.admin import NewTopic
|
||||
from kafka.protocol.admin import DescribeGroupsResponse_v1, DescribeGroupsRequest_v1
|
||||
from kafka.protocol.group import MemberAssignment
|
||||
import socket
|
||||
|
||||
cluster = ClickHouseCluster(__file__)
|
||||
instance = cluster.add_instance('instance',
|
||||
main_configs=['configs/kafka.xml', 'configs/log_conf.xml' ],
|
||||
with_kerberized_kafka=True,
|
||||
clickhouse_path_dir="clickhouse_path"
|
||||
)
|
||||
kafka_id = '' # instance.cluster.kafka_docker_id
|
||||
|
||||
# Helpers
|
||||
|
||||
def check_kafka_is_available():
|
||||
|
||||
# plaintext
|
||||
p = subprocess.Popen(('docker',
|
||||
'exec',
|
||||
'-i',
|
||||
kafka_id,
|
||||
'/usr/bin/kafka-broker-api-versions',
|
||||
'--bootstrap-server',
|
||||
'localhost:9093'),
|
||||
stdout=subprocess.PIPE)
|
||||
p.communicate()
|
||||
return p.returncode == 0
|
||||
|
||||
|
||||
def wait_kafka_is_available(max_retries=50):
|
||||
retries = 0
|
||||
while True:
|
||||
if check_kafka_is_available():
|
||||
break
|
||||
else:
|
||||
retries += 1
|
||||
if retries > max_retries:
|
||||
raise "Kafka is not available"
|
||||
print("Waiting for Kafka to start up")
|
||||
time.sleep(1)
|
||||
|
||||
|
||||
def kafka_produce(topic, messages, timestamp=None):
|
||||
producer = KafkaProducer(bootstrap_servers="localhost:9093")
|
||||
for message in messages:
|
||||
producer.send(topic=topic, value=message, timestamp_ms=timestamp)
|
||||
producer.flush()
|
||||
print ("Produced {} messages for topic {}".format(len(messages), topic))
|
||||
|
||||
|
||||
|
||||
# Fixtures
|
||||
|
||||
@pytest.fixture(scope="module")
|
||||
def kafka_cluster():
|
||||
try:
|
||||
global kafka_id
|
||||
cluster.start()
|
||||
kafka_id = instance.cluster.kerberized_kafka_docker_id
|
||||
print("kafka_id is {}".format(kafka_id))
|
||||
yield cluster
|
||||
|
||||
finally:
|
||||
cluster.shutdown()
|
||||
|
||||
|
||||
@pytest.fixture(autouse=True)
|
||||
def kafka_setup_teardown():
|
||||
instance.query('DROP DATABASE IF EXISTS test; CREATE DATABASE test;')
|
||||
wait_kafka_is_available()
|
||||
print("kafka is available - running test")
|
||||
yield # run test
|
||||
|
||||
# Tests
|
||||
|
||||
@pytest.mark.timeout(180) # wait to build containers
|
||||
def test_kafka_json_as_string(kafka_cluster):
|
||||
kafka_produce('kafka_json_as_string', ['{"t": 123, "e": {"x": "woof"} }', '', '{"t": 124, "e": {"x": "test"} }', '{"F1":"V1","F2":{"F21":"V21","F22":{},"F23":"V23","F24":"2019-12-24T16:28:04"},"F3":"V3"}'])
|
||||
|
||||
instance.query('''
|
||||
CREATE TABLE test.kafka (field String)
|
||||
ENGINE = Kafka
|
||||
SETTINGS kafka_broker_list = 'kerberized_kafka1:19092',
|
||||
kafka_topic_list = 'kafka_json_as_string',
|
||||
kafka_group_name = 'kafka_json_as_string',
|
||||
kafka_format = 'JSONAsString',
|
||||
kafka_flush_interval_ms=1000;
|
||||
''')
|
||||
|
||||
result = instance.query('SELECT * FROM test.kafka;')
|
||||
expected = '''\
|
||||
{"t": 123, "e": {"x": "woof"} }
|
||||
{"t": 124, "e": {"x": "test"} }
|
||||
{"F1":"V1","F2":{"F21":"V21","F22":{},"F23":"V23","F24":"2019-12-24T16:28:04"},"F3":"V3"}
|
||||
'''
|
||||
assert TSV(result) == TSV(expected)
|
||||
assert instance.contains_in_log("Parsing of message (topic: kafka_json_as_string, partition: 0, offset: 1) return no rows")
|
||||
|
||||
def test_kafka_json_as_string_no_kdc(kafka_cluster):
|
||||
kafka_produce('kafka_json_as_string_no_kdc', ['{"t": 123, "e": {"x": "woof"} }', '', '{"t": 124, "e": {"x": "test"} }', '{"F1":"V1","F2":{"F21":"V21","F22":{},"F23":"V23","F24":"2019-12-24T16:28:04"},"F3":"V3"}'])
|
||||
|
||||
kafka_cluster.pause_container('kafka_kerberos')
|
||||
time.sleep(45) # wait for ticket expiration
|
||||
|
||||
instance.query('''
|
||||
CREATE TABLE test.kafka_no_kdc (field String)
|
||||
ENGINE = Kafka
|
||||
SETTINGS kafka_broker_list = 'kerberized_kafka1:19092',
|
||||
kafka_topic_list = 'kafka_json_as_string_no_kdc',
|
||||
kafka_group_name = 'kafka_json_as_string_no_kdc',
|
||||
kafka_format = 'JSONAsString',
|
||||
kafka_flush_interval_ms=1000;
|
||||
''')
|
||||
|
||||
result = instance.query('SELECT * FROM test.kafka_no_kdc;')
|
||||
expected = ''
|
||||
|
||||
kafka_cluster.unpause_container('kafka_kerberos')
|
||||
|
||||
|
||||
assert TSV(result) == TSV(expected)
|
||||
assert instance.contains_in_log("StorageKafka (kafka_no_kdc): Nothing to commit")
|
||||
assert instance.contains_in_log("Ticket expired")
|
||||
assert instance.contains_in_log("Kerberos ticket refresh failed")
|
||||
|
||||
|
||||
if __name__ == '__main__':
|
||||
cluster.start()
|
||||
raw_input("Cluster created, press any key to destroy...")
|
||||
cluster.shutdown()
|
@ -1,11 +1,11 @@
|
||||
<test max_ignored_relative_change="0.15">
|
||||
<settings>
|
||||
<max_memory_usage>10G</max_memory_usage>
|
||||
<max_memory_usage>15G</max_memory_usage>
|
||||
</settings>
|
||||
|
||||
<create_query>CREATE TABLE t (x UInt64, d32 Decimal32(3), d64 Decimal64(4), d128 Decimal128(5)) ENGINE = Memory</create_query>
|
||||
<!-- use less threads to save memory -->
|
||||
<fill_query>INSERT INTO t SELECT number AS x, x % 1000000 AS d32, x AS d64, x d128 FROM numbers_mt(25000000) SETTINGS max_threads = 8</fill_query>
|
||||
<fill_query>INSERT INTO t SELECT number AS x, x % 1000000 AS d32, x AS d64, x d128 FROM numbers_mt(100000000) SETTINGS max_threads = 8</fill_query>
|
||||
<drop_query>DROP TABLE IF EXISTS t</drop_query>
|
||||
|
||||
<query>SELECT toUInt32(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
@ -13,8 +13,8 @@
|
||||
<query>SELECT toInt64(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
<query>SELECT toUInt64(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
<query>SELECT toInt128(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
<query>SELECT toInt256(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
<query>SELECT toUInt256(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
<query>SELECT toInt256(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t LIMIT 10000000 FORMAT Null</query>
|
||||
<query>SELECT toUInt256(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t LIMIT 10000000 FORMAT Null</query>
|
||||
<query>SELECT toFloat32(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
<query>SELECT toFloat64(x) y, toDecimal32(y, 1), toDecimal64(y, 5), toDecimal128(y, 6) FROM t FORMAT Null</query>
|
||||
|
@ -0,0 +1 @@
|
||||
ApplicationA 2020-01-01
|
@ -0,0 +1,31 @@
|
||||
DROP TABLE IF EXISTS APPLICATION;
|
||||
DROP TABLE IF EXISTS DATABASE_IO;
|
||||
|
||||
CREATE TABLE APPLICATION (
|
||||
`Name` LowCardinality(String),
|
||||
`Base` LowCardinality(String)
|
||||
) ENGINE = Memory();
|
||||
|
||||
insert into table APPLICATION values ('ApplicationA', 'BaseA'), ('ApplicationB', 'BaseB') , ('ApplicationC', 'BaseC');
|
||||
|
||||
CREATE TABLE DATABASE_IO (
|
||||
`Application` LowCardinality(String),
|
||||
`Base` LowCardinality(String),
|
||||
`Date` DateTime,
|
||||
`Ios` UInt32 )
|
||||
ENGINE = MergeTree()
|
||||
ORDER BY Date;
|
||||
|
||||
insert into table DATABASE_IO values ('AppA', 'BaseA', '2020-01-01 00:00:00', 1000);
|
||||
|
||||
SELECT `APPLICATION`.`Name` AS `App`,
|
||||
CAST(CAST(`DATABASE_IO`.`Date` AS DATE) AS DATE) AS `date`
|
||||
FROM `DATABASE_IO`
|
||||
INNER
|
||||
JOIN `APPLICATION` ON (`DATABASE_IO`.`Base` = `APPLICATION`.`Base`)
|
||||
WHERE (
|
||||
CAST(CAST(`DATABASE_IO`.`Date` AS DATE) AS TIMESTAMP) >= toDateTime('2020-01-01 00:00:00')
|
||||
);
|
||||
|
||||
DROP TABLE APPLICATION;
|
||||
DROP TABLE DATABASE_IO;
|
3
tests/queries/0_stateless/01499_log_deadlock.reference
Normal file
3
tests/queries/0_stateless/01499_log_deadlock.reference
Normal file
@ -0,0 +1,3 @@
|
||||
6
|
||||
6
|
||||
6
|
26
tests/queries/0_stateless/01499_log_deadlock.sql
Normal file
26
tests/queries/0_stateless/01499_log_deadlock.sql
Normal file
@ -0,0 +1,26 @@
|
||||
DROP TABLE IF EXISTS t;
|
||||
CREATE TABLE t (x UInt8) ENGINE = TinyLog;
|
||||
|
||||
INSERT INTO t VALUES (1), (2), (3);
|
||||
INSERT INTO t SELECT * FROM t;
|
||||
SELECT count() FROM t;
|
||||
|
||||
DROP TABLE t;
|
||||
|
||||
|
||||
CREATE TABLE t (x UInt8) ENGINE = Log;
|
||||
|
||||
INSERT INTO t VALUES (1), (2), (3);
|
||||
INSERT INTO t SELECT * FROM t;
|
||||
SELECT count() FROM t;
|
||||
|
||||
DROP TABLE t;
|
||||
|
||||
|
||||
CREATE TABLE t (x UInt8) ENGINE = StripeLog;
|
||||
|
||||
INSERT INTO t VALUES (1), (2), (3);
|
||||
INSERT INTO t SELECT * FROM t;
|
||||
SELECT count() FROM t;
|
||||
|
||||
DROP TABLE t;
|
@ -0,0 +1,6 @@
|
||||
Testing TinyLog
|
||||
Done TinyLog
|
||||
Testing StripeLog
|
||||
Done StripeLog
|
||||
Testing Log
|
||||
Done Log
|
85
tests/queries/0_stateless/01502_long_log_tinylog_deadlock_race.sh
Executable file
85
tests/queries/0_stateless/01502_long_log_tinylog_deadlock_race.sh
Executable file
@ -0,0 +1,85 @@
|
||||
#!/usr/bin/env bash
|
||||
|
||||
set -e
|
||||
|
||||
CLICKHOUSE_CLIENT_SERVER_LOGS_LEVEL=fatal
|
||||
|
||||
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
. "$CURDIR"/../shell_config.sh
|
||||
|
||||
|
||||
function thread_create {
|
||||
while true; do
|
||||
$CLICKHOUSE_CLIENT --query "CREATE TABLE IF NOT EXISTS $1 (x UInt64, s Array(Nullable(String))) ENGINE = $2" 2>&1 | grep -v -F 'Received exception from server' | grep -v -P 'Code: (60|57)'
|
||||
sleep 0.0$RANDOM
|
||||
done
|
||||
}
|
||||
|
||||
function thread_drop {
|
||||
while true; do
|
||||
$CLICKHOUSE_CLIENT --query "DROP TABLE IF EXISTS $1" 2>&1 | grep -v -F 'Received exception from server' | grep -v -P 'Code: (60|57)'
|
||||
sleep 0.0$RANDOM
|
||||
done
|
||||
}
|
||||
|
||||
function thread_rename {
|
||||
while true; do
|
||||
$CLICKHOUSE_CLIENT --query "RENAME TABLE $1 TO $2" 2>&1 | grep -v -F 'Received exception from server' | grep -v -P 'Code: (60|57)'
|
||||
sleep 0.0$RANDOM
|
||||
done
|
||||
}
|
||||
|
||||
function thread_select {
|
||||
while true; do
|
||||
$CLICKHOUSE_CLIENT --query "SELECT * FROM $1 FORMAT Null" 2>&1 | grep -v -F 'Received exception from server' | grep -v -P 'Code: (60|218)'
|
||||
sleep 0.0$RANDOM
|
||||
done
|
||||
}
|
||||
|
||||
function thread_insert {
|
||||
while true; do
|
||||
$CLICKHOUSE_CLIENT --query "INSERT INTO $1 SELECT rand64(1), [toString(rand64(2))] FROM numbers($2)" 2>&1 | grep -v -F 'Received exception from server' | grep -v -P 'Code: (60|218)'
|
||||
sleep 0.0$RANDOM
|
||||
done
|
||||
}
|
||||
|
||||
function thread_insert_select {
|
||||
while true; do
|
||||
$CLICKHOUSE_CLIENT --query "INSERT INTO $1 SELECT * FROM $2" 2>&1 | grep -v -F 'Received exception from server' | grep -v -P 'Code: (60|218)'
|
||||
sleep 0.0$RANDOM
|
||||
done
|
||||
}
|
||||
|
||||
export -f thread_create
|
||||
export -f thread_drop
|
||||
export -f thread_rename
|
||||
export -f thread_select
|
||||
export -f thread_insert
|
||||
export -f thread_insert_select
|
||||
|
||||
|
||||
# Do randomized queries and expect nothing extraordinary happens.
|
||||
|
||||
function test_with_engine {
|
||||
echo "Testing $1"
|
||||
|
||||
timeout 10 bash -c "thread_create t1 $1" &
|
||||
timeout 10 bash -c "thread_create t2 $1" &
|
||||
timeout 10 bash -c 'thread_drop t1' &
|
||||
timeout 10 bash -c 'thread_drop t2' &
|
||||
timeout 10 bash -c 'thread_rename t1 t2' &
|
||||
timeout 10 bash -c 'thread_rename t2 t1' &
|
||||
timeout 10 bash -c 'thread_select t1' &
|
||||
timeout 10 bash -c 'thread_select t2' &
|
||||
timeout 10 bash -c 'thread_insert t1 5' &
|
||||
timeout 10 bash -c 'thread_insert t2 10' &
|
||||
timeout 10 bash -c 'thread_insert_select t1 t2' &
|
||||
timeout 10 bash -c 'thread_insert_select t2 t1' &
|
||||
|
||||
wait
|
||||
echo "Done $1"
|
||||
}
|
||||
|
||||
test_with_engine TinyLog
|
||||
test_with_engine StripeLog
|
||||
test_with_engine Log
|
Some files were not shown because too many files have changed in this diff Show More
Loading…
Reference in New Issue
Block a user