mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-22 23:52:03 +00:00
Merge branch 'master' into ci_fast_t_in_mq
This commit is contained in:
commit
707c43d17a
@ -25,7 +25,7 @@ CREATE TABLE [IF NOT EXISTS] [db.]table_name [ON CLUSTER cluster]
|
||||
[ORDER BY expr]
|
||||
[PRIMARY KEY expr]
|
||||
[SAMPLE BY expr]
|
||||
[SETTINGS name=value, clean_deleted_rows=value, ...]
|
||||
[SETTINGS name=value, ...]
|
||||
```
|
||||
|
||||
For a description of request parameters, see [statement description](../../../sql-reference/statements/create/table.md).
|
||||
@ -97,7 +97,7 @@ SELECT * FROM mySecondReplacingMT FINAL;
|
||||
:::note
|
||||
`is_deleted` can only be enabled when `ver` is used.
|
||||
|
||||
The row is deleted when `OPTIMIZE ... FINAL CLEANUP` or `OPTIMIZE ... FINAL` is used, or if the engine setting `clean_deleted_rows` has been set to `Always`.
|
||||
The row is deleted when `OPTIMIZE ... FINAL CLEANUP` or `OPTIMIZE ... FINAL` is used.
|
||||
|
||||
No matter the operation on the data, the version must be increased. If two inserted rows have the same version number, the last inserted row is the one kept.
|
||||
|
||||
|
@ -852,16 +852,6 @@ If the file name for column is too long (more than `max_file_name_length` bytes)
|
||||
|
||||
The maximal length of the file name to keep it as is without hashing. Takes effect only if setting `replace_long_file_name_to_hash` is enabled. The value of this setting does not include the length of file extension. So, it is recommended to set it below the maximum filename length (usually 255 bytes) with some gap to avoid filesystem errors. Default value: 127.
|
||||
|
||||
## clean_deleted_rows
|
||||
|
||||
Enable/disable automatic deletion of rows flagged as `is_deleted` when perform `OPTIMIZE ... FINAL` on a table using the ReplacingMergeTree engine. When disabled, the `CLEANUP` keyword has to be added to the `OPTIMIZE ... FINAL` to have the same behaviour.
|
||||
|
||||
Possible values:
|
||||
|
||||
- `Always` or `Never`.
|
||||
|
||||
Default value: `Never`
|
||||
|
||||
## allow_experimental_block_number_column
|
||||
|
||||
Persists virtual column `_block_number` on merges.
|
||||
|
@ -3,7 +3,7 @@ slug: /en/operations/system-tables/asynchronous_metric_log
|
||||
---
|
||||
# asynchronous_metric_log
|
||||
|
||||
Contains the historical values for `system.asynchronous_metrics`, which are saved once per minute. Enabled by default.
|
||||
Contains the historical values for `system.asynchronous_metrics`, which are saved once per time interval (one second by default). Enabled by default.
|
||||
|
||||
Columns:
|
||||
|
||||
|
@ -5,7 +5,7 @@ sidebar_position: 106
|
||||
|
||||
# argMax
|
||||
|
||||
Calculates the `arg` value for a maximum `val` value. If there are several different values of `arg` for maximum values of `val`, returns the first of these values encountered.
|
||||
Calculates the `arg` value for a maximum `val` value. If there are multiple rows with equal `val` being the maximum, which of the associated `arg` is returned is not deterministic.
|
||||
Both parts the `arg` and the `max` behave as [aggregate functions](/docs/en/sql-reference/aggregate-functions/index.md), they both [skip `Null`](/docs/en/sql-reference/aggregate-functions/index.md#null-processing) during processing and return not `Null` values if not `Null` values are available.
|
||||
|
||||
**Syntax**
|
||||
|
@ -5,7 +5,7 @@ sidebar_position: 105
|
||||
|
||||
# argMin
|
||||
|
||||
Calculates the `arg` value for a minimum `val` value. If there are several different values of `arg` for minimum values of `val`, returns the first of these values encountered.
|
||||
Calculates the `arg` value for a minimum `val` value. If there are multiple rows with equal `val` being the maximum, which of the associated `arg` is returned is not deterministic.
|
||||
Both parts the `arg` and the `min` behave as [aggregate functions](/docs/en/sql-reference/aggregate-functions/index.md), they both [skip `Null`](/docs/en/sql-reference/aggregate-functions/index.md#null-processing) during processing and return not `Null` values if not `Null` values are available.
|
||||
|
||||
**Syntax**
|
||||
|
@ -675,7 +675,7 @@ There are two variations of this function:
|
||||
|
||||
Signature:
|
||||
|
||||
For `x` equal to one of the elements in `array_from`, the function returns the corresponding element in `array_to`, i.e. the one at the same array index. Otherwise, it returns `default`. If multiple matching elements exist `array_from`, an arbitrary corresponding element from `array_to` is returned.
|
||||
For `x` equal to one of the elements in `array_from`, the function returns the corresponding element in `array_to`, i.e. the one at the same array index. Otherwise, it returns `default`. If multiple matching elements exist `array_from`, it returns the element corresponding to the first of them.
|
||||
|
||||
`transform(T, Array(T), Array(U), U) -> U`
|
||||
|
||||
|
@ -970,7 +970,7 @@ If the haystack or the LIKE expression are not valid UTF-8, the behavior is unde
|
||||
|
||||
No automatic Unicode normalization is performed, you can use the [normalizeUTF8*()](https://clickhouse.com/docs/en/sql-reference/functions/string-functions/) functions for that.
|
||||
|
||||
To match against literal `%`, `_` and `/` (which are LIKE metacharacters), prepend them with a backslash: `\%`, `\_` and `\\`.
|
||||
To match against literal `%`, `_` and `\` (which are LIKE metacharacters), prepend them with a backslash: `\%`, `\_` and `\\`.
|
||||
The backslash loses its special meaning (i.e. is interpreted literally) if it prepends a character different than `%`, `_` or `\`.
|
||||
Note that ClickHouse requires backslashes in strings [to be quoted as well](../syntax.md#string), so you would actually need to write `\\%`, `\\_` and `\\\\`.
|
||||
|
||||
@ -1768,4 +1768,4 @@ SELECT hasTokenCaseInsensitiveOrNull('Hello World','hello,world');
|
||||
|
||||
```response
|
||||
null
|
||||
```
|
||||
```
|
||||
|
@ -133,8 +133,6 @@ For the query to run successfully, the following conditions must be met:
|
||||
- Both tables must have the same indices and projections.
|
||||
- Both tables must have the same storage policy.
|
||||
|
||||
If both tables have the same storage policy, use hardlink to attach partition. Otherwise, use copying the data to attach partition.
|
||||
|
||||
## REPLACE PARTITION
|
||||
|
||||
``` sql
|
||||
|
@ -99,10 +99,9 @@ SELECT * FROM mySecondReplacingMT FINAL;
|
||||
|
||||
- при использовании инструкции `OPTIMIZE ... FINAL CLEANUP`
|
||||
- при использовании инструкции `OPTIMIZE ... FINAL`
|
||||
- параметр движка `clean_deleted_rows` установлен в значение `Always` (по умолчанию - `Never`)
|
||||
- есть новые версии строки
|
||||
|
||||
Не рекомендуется выполнять `FINAL CLEANUP` или использовать параметр движка `clean_deleted_rows` со значением `Always`, это может привести к неожиданным результатам, например удаленные строки могут вновь появиться.
|
||||
Не рекомендуется выполнять `FINAL CLEANUP`, это может привести к неожиданным результатам, например удаленные строки могут вновь появиться.
|
||||
|
||||
Вне зависимости от производимых изменений над данными, версия должна увеличиваться. Если у двух строк одна и та же версия, то остается только последняя вставленная строка.
|
||||
:::
|
||||
|
@ -5,7 +5,7 @@ sidebar_position: 106
|
||||
|
||||
# argMax {#agg-function-argmax}
|
||||
|
||||
Вычисляет значение `arg` при максимальном значении `val`. Если есть несколько разных значений `arg` для максимальных значений `val`, возвращает первое попавшееся из таких значений.
|
||||
Вычисляет значение `arg` при максимальном значении `val`.
|
||||
|
||||
**Синтаксис**
|
||||
|
||||
|
@ -5,7 +5,7 @@ sidebar_position: 105
|
||||
|
||||
# argMin {#agg-function-argmin}
|
||||
|
||||
Вычисляет значение `arg` при минимальном значении `val`. Если есть несколько разных значений `arg` для минимальных значений `val`, возвращает первое попавшееся из таких значений.
|
||||
Вычисляет значение `arg` при минимальном значении `val`.
|
||||
|
||||
**Синтаксис**
|
||||
|
||||
|
@ -5,7 +5,7 @@ sidebar_position: 106
|
||||
|
||||
# argMax {#agg-function-argmax}
|
||||
|
||||
计算 `val` 最大值对应的 `arg` 值。 如果 `val` 最大值存在几个不同的 `arg` 值,输出遇到的第一个值。
|
||||
计算 `val` 最大值对应的 `arg` 值。
|
||||
|
||||
**语法**
|
||||
|
||||
|
@ -7,7 +7,7 @@ sidebar_position: 105
|
||||
|
||||
语法: `argMin(arg, val)` 或 `argMin(tuple(arg, val))`
|
||||
|
||||
计算 `val` 最小值对应的 `arg` 值。 如果 `val` 最小值存在几个不同的 `arg` 值,输出遇到的第一个(`arg`)值。
|
||||
计算 `val` 最小值对应的 `arg` 值。
|
||||
|
||||
**示例:**
|
||||
|
||||
|
@ -482,6 +482,7 @@ void Client::connect()
|
||||
|
||||
server_version = toString(server_version_major) + "." + toString(server_version_minor) + "." + toString(server_version_patch);
|
||||
load_suggestions = is_interactive && (server_revision >= Suggest::MIN_SERVER_REVISION) && !config().getBool("disable_suggestion", false);
|
||||
wait_for_suggestions_to_load = config().getBool("wait_for_suggestions_to_load", false);
|
||||
|
||||
if (server_display_name = connection->getServerDisplayName(connection_parameters.timeouts); server_display_name.empty())
|
||||
server_display_name = config().getString("host", "localhost");
|
||||
|
@ -572,6 +572,7 @@ void LocalServer::processConfig()
|
||||
const std::string clickhouse_dialect{"clickhouse"};
|
||||
load_suggestions = (is_interactive || delayed_interactive) && !config().getBool("disable_suggestion", false)
|
||||
&& config().getString("dialect", clickhouse_dialect) == clickhouse_dialect;
|
||||
wait_for_suggestions_to_load = config().getBool("wait_for_suggestions_to_load", false);
|
||||
|
||||
auto logging = (config().has("logger.console")
|
||||
|| config().has("logger.level")
|
||||
@ -847,6 +848,8 @@ void LocalServer::processOptions(const OptionsDescription &, const CommandLineOp
|
||||
config().setString("logger.level", options["logger.level"].as<std::string>());
|
||||
if (options.count("send_logs_level"))
|
||||
config().setString("send_logs_level", options["send_logs_level"].as<std::string>());
|
||||
if (options.count("wait_for_suggestions_to_load"))
|
||||
config().setBool("wait_for_suggestions_to_load", true);
|
||||
}
|
||||
|
||||
void LocalServer::readArguments(int argc, char ** argv, Arguments & common_arguments, std::vector<Arguments> &, std::vector<Arguments> &)
|
||||
|
@ -19,6 +19,19 @@ namespace ErrorCodes
|
||||
extern const int LOGICAL_ERROR;
|
||||
}
|
||||
|
||||
using namespace std::literals;
|
||||
static constexpr std::array boolean_functions{
|
||||
"equals"sv, "notEquals"sv, "less"sv, "greaterOrEquals"sv, "greater"sv, "lessOrEquals"sv, "in"sv, "notIn"sv,
|
||||
"globalIn"sv, "globalNotIn"sv, "nullIn"sv, "notNullIn"sv, "globalNullIn"sv, "globalNullNotIn"sv, "isNull"sv, "isNotNull"sv,
|
||||
"like"sv, "notLike"sv, "ilike"sv, "notILike"sv, "empty"sv, "notEmpty"sv, "not"sv, "and"sv,
|
||||
"or"sv};
|
||||
|
||||
static bool isBooleanFunction(const String & func_name)
|
||||
{
|
||||
return std::any_of(
|
||||
boolean_functions.begin(), boolean_functions.end(), [&](const auto boolean_func) { return func_name == boolean_func; });
|
||||
}
|
||||
|
||||
/// Visitor that optimizes logical expressions _only_ in JOIN ON section
|
||||
class JoinOnLogicalExpressionOptimizerVisitor : public InDepthQueryTreeVisitorWithContext<JoinOnLogicalExpressionOptimizerVisitor>
|
||||
{
|
||||
@ -270,6 +283,12 @@ public:
|
||||
tryOptimizeAndEqualsNotEqualsChain(node);
|
||||
return;
|
||||
}
|
||||
|
||||
if (function_node->getFunctionName() == "equals")
|
||||
{
|
||||
tryOptimizeOutRedundantEquals(node);
|
||||
return;
|
||||
}
|
||||
}
|
||||
|
||||
private:
|
||||
@ -569,6 +588,56 @@ private:
|
||||
function_node.getArguments().getNodes() = std::move(or_operands);
|
||||
function_node.resolveAsFunction(or_function_resolver);
|
||||
}
|
||||
|
||||
void tryOptimizeOutRedundantEquals(QueryTreeNodePtr & node)
|
||||
{
|
||||
auto & function_node = node->as<FunctionNode &>();
|
||||
assert(function_node.getFunctionName() == "equals");
|
||||
|
||||
const auto function_arguments = function_node.getArguments().getNodes();
|
||||
if (function_arguments.size() != 2)
|
||||
return;
|
||||
|
||||
const auto & lhs = function_arguments[0];
|
||||
const auto & rhs = function_arguments[1];
|
||||
|
||||
UInt64 constant_value;
|
||||
bool is_lhs_const;
|
||||
if (const auto * lhs_constant = lhs->as<ConstantNode>())
|
||||
{
|
||||
if (!lhs_constant->getValue().tryGet<UInt64>(constant_value) || constant_value > 1
|
||||
|| isNullableOrLowCardinalityNullable(lhs_constant->getResultType()))
|
||||
return;
|
||||
is_lhs_const = true;
|
||||
}
|
||||
else if (const auto * rhs_constant = rhs->as<ConstantNode>())
|
||||
{
|
||||
if (!rhs_constant->getValue().tryGet<UInt64>(constant_value) || constant_value > 1
|
||||
|| isNullableOrLowCardinalityNullable(rhs_constant->getResultType()))
|
||||
return;
|
||||
is_lhs_const = false;
|
||||
}
|
||||
else
|
||||
return;
|
||||
|
||||
const FunctionNode * child_function = is_lhs_const ? rhs->as<FunctionNode>() : lhs->as<FunctionNode>();
|
||||
if (!child_function || !isBooleanFunction(child_function->getFunctionName()))
|
||||
return;
|
||||
|
||||
// if we have something like `function = 0`, we need to add a `NOT` when dropping the `= 0`
|
||||
if (constant_value == 0)
|
||||
{
|
||||
auto not_resolver = FunctionFactory::instance().get("not", getContext());
|
||||
const auto not_node = std::make_shared<FunctionNode>("not");
|
||||
auto & arguments = not_node->getArguments().getNodes();
|
||||
arguments.reserve(1);
|
||||
arguments.push_back(is_lhs_const ? rhs : lhs);
|
||||
not_node->resolveAsFunction(not_resolver->build(not_node->getArgumentColumns()));
|
||||
node = not_node;
|
||||
}
|
||||
else
|
||||
node = is_lhs_const ? rhs : lhs;
|
||||
}
|
||||
};
|
||||
|
||||
void LogicalExpressionOptimizerPass::run(QueryTreeNodePtr & query_tree_node, ContextPtr context)
|
||||
|
@ -96,6 +96,18 @@ namespace DB
|
||||
*
|
||||
* SELECT * FROM t1 JOIN t2 ON a <=> b
|
||||
* -------------------------------
|
||||
*
|
||||
* 7. Remove redundant equality checks on boolean functions.
|
||||
* - these requndant checks cause the primary index to not be used when if the query involves any primary key columns
|
||||
* -------------------------------
|
||||
* SELECT * FROM t1 WHERE a IN (n) = 1
|
||||
* SELECT * FROM t1 WHERE a IN (n) = 0
|
||||
*
|
||||
* will be transformed into
|
||||
*
|
||||
* SELECT * FROM t1 WHERE a IN (n)
|
||||
* SELECT * FROM t1 WHERE NOT a IN (n)
|
||||
* -------------------------------
|
||||
*/
|
||||
|
||||
class LogicalExpressionOptimizerPass final : public IQueryTreePass
|
||||
|
@ -50,44 +50,20 @@ BackupReaderAzureBlobStorage::~BackupReaderAzureBlobStorage() = default;
|
||||
|
||||
bool BackupReaderAzureBlobStorage::fileExists(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
return object_storage->exists(StoredObject(key));
|
||||
}
|
||||
|
||||
UInt64 BackupReaderAzureBlobStorage::getFileSize(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
ObjectMetadata object_metadata = object_storage->getObjectMetadata(key);
|
||||
return object_metadata.size_bytes;
|
||||
}
|
||||
|
||||
std::unique_ptr<SeekableReadBuffer> BackupReaderAzureBlobStorage::readFile(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
return std::make_unique<ReadBufferFromAzureBlobStorage>(
|
||||
client, key, read_settings, settings->max_single_read_retries,
|
||||
settings->max_single_download_retries);
|
||||
@ -194,7 +170,7 @@ void BackupWriterAzureBlobStorage::copyFile(const String & destination, const St
|
||||
client,
|
||||
client,
|
||||
configuration.container,
|
||||
fs::path(source),
|
||||
fs::path(configuration.blob_path)/ source,
|
||||
0,
|
||||
size,
|
||||
/* dest_container */ configuration.container,
|
||||
@ -207,7 +183,7 @@ void BackupWriterAzureBlobStorage::copyFile(const String & destination, const St
|
||||
|
||||
void BackupWriterAzureBlobStorage::copyDataToFile(const String & path_in_backup, const CreateReadBufferFunction & create_read_buffer, UInt64 start_pos, UInt64 length)
|
||||
{
|
||||
copyDataToAzureBlobStorageFile(create_read_buffer, start_pos, length, client, configuration.container, path_in_backup, settings,
|
||||
copyDataToAzureBlobStorageFile(create_read_buffer, start_pos, length, client, configuration.container, fs::path(configuration.blob_path) / path_in_backup, settings,
|
||||
threadPoolCallbackRunner<void>(getBackupsIOThreadPool().get(), "BackupWRAzure"));
|
||||
}
|
||||
|
||||
@ -215,29 +191,13 @@ BackupWriterAzureBlobStorage::~BackupWriterAzureBlobStorage() = default;
|
||||
|
||||
bool BackupWriterAzureBlobStorage::fileExists(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
return object_storage->exists(StoredObject(key));
|
||||
}
|
||||
|
||||
UInt64 BackupWriterAzureBlobStorage::getFileSize(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
RelativePathsWithMetadata children;
|
||||
object_storage->listObjects(key,children,/*max_keys*/0);
|
||||
if (children.empty())
|
||||
@ -247,16 +207,7 @@ UInt64 BackupWriterAzureBlobStorage::getFileSize(const String & file_name)
|
||||
|
||||
std::unique_ptr<ReadBuffer> BackupWriterAzureBlobStorage::readFile(const String & file_name, size_t /*expected_file_size*/)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
return std::make_unique<ReadBufferFromAzureBlobStorage>(
|
||||
client, key, read_settings, settings->max_single_read_retries,
|
||||
settings->max_single_download_retries);
|
||||
@ -264,15 +215,7 @@ std::unique_ptr<ReadBuffer> BackupWriterAzureBlobStorage::readFile(const String
|
||||
|
||||
std::unique_ptr<WriteBuffer> BackupWriterAzureBlobStorage::writeFile(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
return std::make_unique<WriteBufferFromAzureBlobStorage>(
|
||||
client,
|
||||
key,
|
||||
@ -283,15 +226,7 @@ std::unique_ptr<WriteBuffer> BackupWriterAzureBlobStorage::writeFile(const Strin
|
||||
|
||||
void BackupWriterAzureBlobStorage::removeFile(const String & file_name)
|
||||
{
|
||||
String key;
|
||||
if (startsWith(file_name, "."))
|
||||
{
|
||||
key= configuration.blob_path + file_name;
|
||||
}
|
||||
else
|
||||
{
|
||||
key = file_name;
|
||||
}
|
||||
String key = fs::path(configuration.blob_path) / file_name;
|
||||
StoredObject object(key);
|
||||
object_storage->removeObjectIfExists(object);
|
||||
}
|
||||
@ -300,7 +235,7 @@ void BackupWriterAzureBlobStorage::removeFiles(const Strings & file_names)
|
||||
{
|
||||
StoredObjects objects;
|
||||
for (const auto & file_name : file_names)
|
||||
objects.emplace_back(file_name);
|
||||
objects.emplace_back(fs::path(configuration.blob_path) / file_name);
|
||||
|
||||
object_storage->removeObjectsIfExist(objects);
|
||||
|
||||
@ -310,7 +245,7 @@ void BackupWriterAzureBlobStorage::removeFilesBatch(const Strings & file_names)
|
||||
{
|
||||
StoredObjects objects;
|
||||
for (const auto & file_name : file_names)
|
||||
objects.emplace_back(file_name);
|
||||
objects.emplace_back(fs::path(configuration.blob_path) / file_name);
|
||||
|
||||
object_storage->removeObjectsIfExist(objects);
|
||||
}
|
||||
|
@ -2064,7 +2064,7 @@ MultiQueryProcessingStage ClientBase::analyzeMultiQueryText(
|
||||
return MultiQueryProcessingStage::QUERIES_END;
|
||||
|
||||
// Remove leading empty newlines and other whitespace, because they
|
||||
// are annoying to filter in query log. This is mostly relevant for
|
||||
// are annoying to filter in the query log. This is mostly relevant for
|
||||
// the tests.
|
||||
while (this_query_begin < all_queries_end && isWhitespaceASCII(*this_query_begin))
|
||||
++this_query_begin;
|
||||
@ -2098,7 +2098,7 @@ MultiQueryProcessingStage ClientBase::analyzeMultiQueryText(
|
||||
is_interactive,
|
||||
ignore_error);
|
||||
}
|
||||
catch (Exception & e)
|
||||
catch (const Exception & e)
|
||||
{
|
||||
current_exception.reset(e.clone());
|
||||
return MultiQueryProcessingStage::PARSING_EXCEPTION;
|
||||
@ -2123,9 +2123,9 @@ MultiQueryProcessingStage ClientBase::analyzeMultiQueryText(
|
||||
// INSERT queries may have the inserted data in the query text
|
||||
// that follow the query itself, e.g. "insert into t format CSV 1;2".
|
||||
// They need special handling. First of all, here we find where the
|
||||
// inserted data ends. In multy-query mode, it is delimited by a
|
||||
// inserted data ends. In multi-query mode, it is delimited by a
|
||||
// newline.
|
||||
// The VALUES format needs even more handling -- we also allow the
|
||||
// The VALUES format needs even more handling - we also allow the
|
||||
// data to be delimited by semicolon. This case is handled later by
|
||||
// the format parser itself.
|
||||
// We can't do multiline INSERTs with inline data, because most
|
||||
@ -2477,9 +2477,9 @@ void ClientBase::runInteractive()
|
||||
{
|
||||
/// Load suggestion data from the server.
|
||||
if (global_context->getApplicationType() == Context::ApplicationType::CLIENT)
|
||||
suggest->load<Connection>(global_context, connection_parameters, config().getInt("suggestion_limit"));
|
||||
suggest->load<Connection>(global_context, connection_parameters, config().getInt("suggestion_limit"), wait_for_suggestions_to_load);
|
||||
else if (global_context->getApplicationType() == Context::ApplicationType::LOCAL)
|
||||
suggest->load<LocalConnection>(global_context, connection_parameters, config().getInt("suggestion_limit"));
|
||||
suggest->load<LocalConnection>(global_context, connection_parameters, config().getInt("suggestion_limit"), wait_for_suggestions_to_load);
|
||||
}
|
||||
|
||||
if (home_path.empty())
|
||||
@ -2975,6 +2975,7 @@ void ClientBase::init(int argc, char ** argv)
|
||||
("progress", po::value<ProgressOption>()->implicit_value(ProgressOption::TTY, "tty")->default_value(ProgressOption::DEFAULT, "default"), "Print progress of queries execution - to TTY: tty|on|1|true|yes; to STDERR non-interactive mode: err; OFF: off|0|false|no; DEFAULT - interactive to TTY, non-interactive is off")
|
||||
|
||||
("disable_suggestion,A", "Disable loading suggestion data. Note that suggestion data is loaded asynchronously through a second connection to ClickHouse server. Also it is reasonable to disable suggestion if you want to paste a query with TAB characters. Shorthand option -A is for those who get used to mysql client.")
|
||||
("wait_for_suggestions_to_load", "Load suggestion data synchonously.")
|
||||
("time,t", "print query execution time to stderr in non-interactive mode (for benchmarks)")
|
||||
|
||||
("echo", "in batch mode, print query before execution")
|
||||
@ -3104,6 +3105,8 @@ void ClientBase::init(int argc, char ** argv)
|
||||
config().setBool("echo", true);
|
||||
if (options.count("disable_suggestion"))
|
||||
config().setBool("disable_suggestion", true);
|
||||
if (options.count("wait_for_suggestions_to_load"))
|
||||
config().setBool("wait_for_suggestions_to_load", true);
|
||||
if (options.count("suggestion_limit"))
|
||||
config().setInt("suggestion_limit", options["suggestion_limit"].as<int>());
|
||||
if (options.count("highlight"))
|
||||
|
@ -209,6 +209,7 @@ protected:
|
||||
|
||||
std::optional<Suggest> suggest;
|
||||
bool load_suggestions = false;
|
||||
bool wait_for_suggestions_to_load = false;
|
||||
|
||||
std::vector<String> queries; /// Queries passed via '--query'
|
||||
std::vector<String> queries_files; /// If not empty, queries will be read from these files
|
||||
|
@ -1,11 +1,14 @@
|
||||
#include "ClientBaseHelpers.h"
|
||||
|
||||
|
||||
#include <Common/DateLUT.h>
|
||||
#include <Common/LocalDate.h>
|
||||
#include <Parsers/Lexer.h>
|
||||
#include <Parsers/ParserQuery.h>
|
||||
#include <Parsers/parseQuery.h>
|
||||
#include <Common/UTF8Helpers.h>
|
||||
|
||||
#include <iostream>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
@ -96,77 +99,103 @@ void highlight(const String & query, std::vector<replxx::Replxx::Color> & colors
|
||||
{
|
||||
using namespace replxx;
|
||||
|
||||
static const std::unordered_map<TokenType, Replxx::Color> token_to_color
|
||||
= {{TokenType::Whitespace, Replxx::Color::DEFAULT},
|
||||
{TokenType::Comment, Replxx::Color::GRAY},
|
||||
{TokenType::BareWord, Replxx::Color::DEFAULT},
|
||||
{TokenType::Number, Replxx::Color::GREEN},
|
||||
{TokenType::StringLiteral, Replxx::Color::CYAN},
|
||||
{TokenType::QuotedIdentifier, Replxx::Color::MAGENTA},
|
||||
{TokenType::OpeningRoundBracket, Replxx::Color::BROWN},
|
||||
{TokenType::ClosingRoundBracket, Replxx::Color::BROWN},
|
||||
{TokenType::OpeningSquareBracket, Replxx::Color::BROWN},
|
||||
{TokenType::ClosingSquareBracket, Replxx::Color::BROWN},
|
||||
{TokenType::DoubleColon, Replxx::Color::BROWN},
|
||||
{TokenType::OpeningCurlyBrace, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::ClosingCurlyBrace, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
/// The `colors` array maps to a Unicode code point position in a string into a color.
|
||||
/// A color is set for every position individually (not for a range).
|
||||
|
||||
{TokenType::Comma, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Semicolon, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::VerticalDelimiter, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Dot, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Asterisk, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::HereDoc, Replxx::Color::CYAN},
|
||||
{TokenType::Plus, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Minus, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Slash, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Percent, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Arrow, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::QuestionMark, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Colon, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Equals, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::NotEquals, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Less, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Greater, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::LessOrEquals, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::GreaterOrEquals, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Spaceship, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::Concatenation, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::At, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{TokenType::DoubleAt, Replxx::Color::MAGENTA},
|
||||
/// Empty input.
|
||||
if (colors.empty())
|
||||
return;
|
||||
|
||||
{TokenType::EndOfStream, Replxx::Color::DEFAULT},
|
||||
/// The colors should be legible (and look gorgeous) in both dark and light themes.
|
||||
/// When modifying this, check it in both themes.
|
||||
|
||||
{TokenType::Error, Replxx::Color::RED},
|
||||
{TokenType::ErrorMultilineCommentIsNotClosed, Replxx::Color::RED},
|
||||
{TokenType::ErrorSingleQuoteIsNotClosed, Replxx::Color::RED},
|
||||
{TokenType::ErrorDoubleQuoteIsNotClosed, Replxx::Color::RED},
|
||||
{TokenType::ErrorSinglePipeMark, Replxx::Color::RED},
|
||||
{TokenType::ErrorWrongNumber, Replxx::Color::RED},
|
||||
{TokenType::ErrorMaxQuerySizeExceeded, Replxx::Color::RED}};
|
||||
|
||||
const Replxx::Color unknown_token_color = Replxx::Color::RED;
|
||||
|
||||
Lexer lexer(query.data(), query.data() + query.size());
|
||||
size_t pos = 0;
|
||||
|
||||
for (Token token = lexer.nextToken(); !token.isEnd(); token = lexer.nextToken())
|
||||
static const std::unordered_map<Highlight, Replxx::Color> type_to_color =
|
||||
{
|
||||
if (token.type == TokenType::Semicolon || token.type == TokenType::VerticalDelimiter)
|
||||
ReplxxLineReader::setLastIsDelimiter(true);
|
||||
else if (token.type != TokenType::Whitespace)
|
||||
ReplxxLineReader::setLastIsDelimiter(false);
|
||||
{Highlight::keyword, replxx::color::bold(Replxx::Color::DEFAULT)},
|
||||
{Highlight::identifier, Replxx::Color::CYAN},
|
||||
{Highlight::function, Replxx::Color::BROWN},
|
||||
{Highlight::alias, replxx::color::rgb666(0, 4, 4)},
|
||||
{Highlight::substitution, Replxx::Color::MAGENTA},
|
||||
{Highlight::number, replxx::color::rgb666(0, 4, 0)},
|
||||
{Highlight::string, Replxx::Color::GREEN},
|
||||
};
|
||||
|
||||
size_t utf8_len = UTF8::countCodePoints(reinterpret_cast<const UInt8 *>(token.begin), token.size());
|
||||
for (size_t code_point_index = 0; code_point_index < utf8_len; ++code_point_index)
|
||||
/// We set reasonably small limits for size/depth, because we don't want the CLI to be slow.
|
||||
/// While syntax highlighting is unneeded for long queries, which the user couldn't read anyway.
|
||||
|
||||
const char * begin = query.data();
|
||||
const char * end = begin + query.size();
|
||||
Tokens tokens(begin, end, 1000, true);
|
||||
IParser::Pos token_iterator(tokens, static_cast<uint32_t>(1000), static_cast<uint32_t>(10000));
|
||||
Expected expected;
|
||||
expected.enable_highlighting = true;
|
||||
|
||||
/// We don't do highlighting for foreign dialects, such as PRQL and Kusto.
|
||||
/// Only normal ClickHouse SQL queries are highlighted.
|
||||
|
||||
/// Currently we highlight only the first query in the multi-query mode.
|
||||
|
||||
ParserQuery parser(end);
|
||||
ASTPtr ast;
|
||||
bool parse_res = false;
|
||||
|
||||
try
|
||||
{
|
||||
parse_res = parser.parse(token_iterator, ast, expected);
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
/// Skip highlighting in the case of exceptions during parsing.
|
||||
/// It is ok to ignore unknown exceptions here.
|
||||
return;
|
||||
}
|
||||
|
||||
size_t pos = 0;
|
||||
const char * prev = begin;
|
||||
for (const auto & range : expected.highlights)
|
||||
{
|
||||
auto it = type_to_color.find(range.highlight);
|
||||
if (it != type_to_color.end())
|
||||
{
|
||||
if (token_to_color.find(token.type) != token_to_color.end())
|
||||
colors[pos + code_point_index] = token_to_color.at(token.type);
|
||||
else
|
||||
colors[pos + code_point_index] = unknown_token_color;
|
||||
}
|
||||
/// We have to map from byte positions to Unicode positions.
|
||||
pos += UTF8::countCodePoints(reinterpret_cast<const UInt8 *>(prev), range.begin - prev);
|
||||
size_t utf8_len = UTF8::countCodePoints(reinterpret_cast<const UInt8 *>(range.begin), range.end - range.begin);
|
||||
|
||||
pos += utf8_len;
|
||||
for (size_t code_point_index = 0; code_point_index < utf8_len; ++code_point_index)
|
||||
colors[pos + code_point_index] = it->second;
|
||||
|
||||
pos += utf8_len;
|
||||
prev = range.end;
|
||||
}
|
||||
}
|
||||
|
||||
Token last_token = token_iterator.max();
|
||||
/// Raw data in INSERT queries, which is not necessarily tokenized.
|
||||
const char * insert_data = ast ? getInsertData(ast) : nullptr;
|
||||
|
||||
/// Highlight the last error in red. If the parser failed or the lexer found an invalid token,
|
||||
/// or if it didn't parse all the data (except, the data for INSERT query, which is legitimately unparsed)
|
||||
if ((!parse_res || last_token.isError() || (!token_iterator->isEnd() && token_iterator->type != TokenType::Semicolon))
|
||||
&& !(insert_data && expected.max_parsed_pos >= insert_data)
|
||||
&& expected.max_parsed_pos >= prev)
|
||||
{
|
||||
pos += UTF8::countCodePoints(reinterpret_cast<const UInt8 *>(prev), expected.max_parsed_pos - prev);
|
||||
|
||||
if (pos >= colors.size())
|
||||
pos = colors.size() - 1;
|
||||
|
||||
colors[pos] = Replxx::Color::BRIGHTRED;
|
||||
}
|
||||
|
||||
/// This is a callback for the client/local app to better find query end. Note: this is a kludge, remove it.
|
||||
if (last_token.type == TokenType::Semicolon || last_token.type == TokenType::VerticalDelimiter
|
||||
|| query.ends_with(';') || query.ends_with("\\G")) /// This is for raw data in INSERT queries, which is not necessarily tokenized.
|
||||
{
|
||||
ReplxxLineReader::setLastIsDelimiter(true);
|
||||
}
|
||||
else if (last_token.type != TokenType::Whitespace)
|
||||
{
|
||||
ReplxxLineReader::setLastIsDelimiter(false);
|
||||
}
|
||||
}
|
||||
#endif
|
||||
|
@ -110,7 +110,7 @@ static String getLoadSuggestionQuery(Int32 suggestion_limit, bool basic_suggesti
|
||||
}
|
||||
|
||||
template <typename ConnectionType>
|
||||
void Suggest::load(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit)
|
||||
void Suggest::load(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit, bool wait_for_load)
|
||||
{
|
||||
loading_thread = std::thread([my_context = Context::createCopy(context), connection_parameters, suggestion_limit, this]
|
||||
{
|
||||
@ -152,6 +152,9 @@ void Suggest::load(ContextPtr context, const ConnectionParameters & connection_p
|
||||
|
||||
/// Note that keyword suggestions are available even if we cannot load data from server.
|
||||
});
|
||||
|
||||
if (wait_for_load)
|
||||
loading_thread.join();
|
||||
}
|
||||
|
||||
void Suggest::load(IServerConnection & connection,
|
||||
@ -228,8 +231,8 @@ void Suggest::fillWordsFromBlock(const Block & block)
|
||||
}
|
||||
|
||||
template
|
||||
void Suggest::load<Connection>(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit);
|
||||
void Suggest::load<Connection>(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit, bool wait_for_load);
|
||||
|
||||
template
|
||||
void Suggest::load<LocalConnection>(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit);
|
||||
void Suggest::load<LocalConnection>(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit, bool wait_for_load);
|
||||
}
|
||||
|
@ -27,7 +27,7 @@ public:
|
||||
|
||||
/// Load suggestions for clickhouse-client.
|
||||
template <typename ConnectionType>
|
||||
void load(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit);
|
||||
void load(ContextPtr context, const ConnectionParameters & connection_parameters, Int32 suggestion_limit, bool wait_for_load);
|
||||
|
||||
void load(IServerConnection & connection,
|
||||
const ConnectionTimeouts & timeouts,
|
||||
|
@ -56,7 +56,7 @@ static std::unique_ptr<ReadBufferFromFilePRead> openFileIfExists(const std::stri
|
||||
|
||||
|
||||
AsynchronousMetrics::AsynchronousMetrics(
|
||||
int update_period_seconds,
|
||||
unsigned update_period_seconds,
|
||||
const ProtocolServerMetricsFunc & protocol_server_metrics_func_)
|
||||
: update_period(update_period_seconds)
|
||||
, log(getLogger("AsynchronousMetrics"))
|
||||
|
@ -44,7 +44,7 @@ struct ProtocolServerMetrics
|
||||
size_t current_threads;
|
||||
};
|
||||
|
||||
/** Periodically (by default, each minute, starting at 30 seconds offset)
|
||||
/** Periodically (by default, each second)
|
||||
* calculates and updates some metrics,
|
||||
* that are not updated automatically (so, need to be asynchronously calculated).
|
||||
*
|
||||
@ -64,7 +64,7 @@ public:
|
||||
using ProtocolServerMetricsFunc = std::function<std::vector<ProtocolServerMetrics>()>;
|
||||
|
||||
AsynchronousMetrics(
|
||||
int update_period_seconds,
|
||||
unsigned update_period_seconds,
|
||||
const ProtocolServerMetricsFunc & protocol_server_metrics_func_);
|
||||
|
||||
virtual ~AsynchronousMetrics();
|
||||
|
@ -296,6 +296,19 @@ public:
|
||||
return it->getMapped();
|
||||
}
|
||||
|
||||
/// Only inserts the value if key isn't already present
|
||||
void ALWAYS_INLINE insertIfNotPresent(const Key & x, const Cell::Mapped & value)
|
||||
{
|
||||
LookupResult it;
|
||||
bool inserted;
|
||||
this->emplace(x, it, inserted);
|
||||
if (inserted)
|
||||
{
|
||||
new (&it->getMapped()) typename Cell::Mapped();
|
||||
it->getMapped() = value;
|
||||
}
|
||||
}
|
||||
|
||||
const typename Cell::Mapped & ALWAYS_INLINE at(const Key & x) const
|
||||
{
|
||||
if (auto it = this->find(x); it != this->end())
|
||||
|
@ -114,7 +114,7 @@ void updateKeeperInformation(KeeperDispatcher & keeper_dispatcher, AsynchronousM
|
||||
}
|
||||
|
||||
KeeperAsynchronousMetrics::KeeperAsynchronousMetrics(
|
||||
ContextPtr context_, int update_period_seconds, const ProtocolServerMetricsFunc & protocol_server_metrics_func_)
|
||||
ContextPtr context_, unsigned update_period_seconds, const ProtocolServerMetricsFunc & protocol_server_metrics_func_)
|
||||
: AsynchronousMetrics(update_period_seconds, protocol_server_metrics_func_), context(std::move(context_))
|
||||
{
|
||||
}
|
||||
|
@ -13,7 +13,7 @@ class KeeperAsynchronousMetrics : public AsynchronousMetrics
|
||||
{
|
||||
public:
|
||||
KeeperAsynchronousMetrics(
|
||||
ContextPtr context_, int update_period_seconds, const ProtocolServerMetricsFunc & protocol_server_metrics_func_);
|
||||
ContextPtr context_, unsigned update_period_seconds, const ProtocolServerMetricsFunc & protocol_server_metrics_func_);
|
||||
~KeeperAsynchronousMetrics() override;
|
||||
|
||||
private:
|
||||
|
@ -3721,8 +3721,23 @@ namespace
|
||||
return std::make_shared<DataTypeEnum<Type>>(std::move(values));
|
||||
}
|
||||
|
||||
std::optional<NameAndTypePair> getNameAndDataTypeFromField(const google::protobuf::FieldDescriptor * field_descriptor, bool skip_unsupported_fields, bool allow_repeat = true)
|
||||
std::optional<NameAndTypePair> getNameAndDataTypeFromField(
|
||||
const google::protobuf::FieldDescriptor * field_descriptor, bool skip_unsupported_fields, bool allow_repeat);
|
||||
|
||||
std::optional<NameAndTypePair> getNameAndDataTypeFromFieldRecursive(
|
||||
const google::protobuf::FieldDescriptor * field_descriptor,
|
||||
bool skip_unsupported_fields,
|
||||
bool allow_repeat,
|
||||
std::unordered_set<const google::protobuf::FieldDescriptor *> & pending_resolution)
|
||||
{
|
||||
if (pending_resolution.contains(field_descriptor))
|
||||
{
|
||||
if (skip_unsupported_fields)
|
||||
return std::nullopt;
|
||||
throw Exception(ErrorCodes::BAD_ARGUMENTS, "ClickHouse doesn't support type recursion ({})", field_descriptor->full_name());
|
||||
}
|
||||
pending_resolution.emplace(field_descriptor);
|
||||
|
||||
if (allow_repeat && field_descriptor->is_map())
|
||||
{
|
||||
auto name_and_type = getNameAndDataTypeFromField(field_descriptor, skip_unsupported_fields, false);
|
||||
@ -3804,7 +3819,8 @@ namespace
|
||||
else if (message_descriptor->field_count() == 1)
|
||||
{
|
||||
const auto * nested_field_descriptor = message_descriptor->field(0);
|
||||
auto nested_name_and_type = getNameAndDataTypeFromField(nested_field_descriptor, skip_unsupported_fields);
|
||||
auto nested_name_and_type
|
||||
= getNameAndDataTypeFromFieldRecursive(nested_field_descriptor, skip_unsupported_fields, true, pending_resolution);
|
||||
if (!nested_name_and_type)
|
||||
return std::nullopt;
|
||||
return NameAndTypePair{field_descriptor->name() + "_" + nested_name_and_type->name, nested_name_and_type->type};
|
||||
@ -3815,7 +3831,8 @@ namespace
|
||||
Strings nested_names;
|
||||
for (int i = 0; i != message_descriptor->field_count(); ++i)
|
||||
{
|
||||
auto nested_name_and_type = getNameAndDataTypeFromField(message_descriptor->field(i), skip_unsupported_fields);
|
||||
auto nested_name_and_type = getNameAndDataTypeFromFieldRecursive(
|
||||
message_descriptor->field(i), skip_unsupported_fields, true, pending_resolution);
|
||||
if (!nested_name_and_type)
|
||||
continue;
|
||||
nested_types.push_back(nested_name_and_type->type);
|
||||
@ -3831,6 +3848,14 @@ namespace
|
||||
|
||||
UNREACHABLE();
|
||||
}
|
||||
|
||||
std::optional<NameAndTypePair> getNameAndDataTypeFromField(
|
||||
const google::protobuf::FieldDescriptor * field_descriptor, bool skip_unsupported_fields, bool allow_repeat = true)
|
||||
{
|
||||
/// Keep track of the fields that are pending resolution to avoid recursive types, which are unsupported
|
||||
std::unordered_set<const google::protobuf::FieldDescriptor *> pending_resolution{};
|
||||
return getNameAndDataTypeFromFieldRecursive(field_descriptor, skip_unsupported_fields, allow_repeat, pending_resolution);
|
||||
}
|
||||
}
|
||||
|
||||
std::unique_ptr<ProtobufSerializer> ProtobufSerializer::create(
|
||||
|
@ -755,7 +755,6 @@ namespace
|
||||
|
||||
WhichDataType which(from_type);
|
||||
|
||||
/// Note: Doesn't check the duplicates in the `from` array.
|
||||
/// Field may be of Float type, but for the purpose of bitwise equality we can treat them as UInt64
|
||||
if (isNativeNumber(which) || which.isDecimal32() || which.isDecimal64() || which.isEnum())
|
||||
{
|
||||
@ -777,7 +776,7 @@ namespace
|
||||
#pragma clang diagnostic pop
|
||||
|
||||
memcpy(dst, ref.data, ref.size);
|
||||
table[key] = i;
|
||||
table.insertIfNotPresent(key, i);
|
||||
}
|
||||
}
|
||||
}
|
||||
@ -790,7 +789,7 @@ namespace
|
||||
if (applyVisitor(FieldVisitorAccurateEquals(), (*cache.from_column)[i], (*from_column_uncasted)[i]))
|
||||
{
|
||||
StringRef ref = cache.from_column->getDataAt(i);
|
||||
table[ref] = i;
|
||||
table.insertIfNotPresent(ref, i);
|
||||
}
|
||||
}
|
||||
}
|
||||
@ -804,7 +803,7 @@ namespace
|
||||
{
|
||||
SipHash hash;
|
||||
cache.from_column->updateHashWithValue(i, hash);
|
||||
table[hash.get128()] = i;
|
||||
table.insertIfNotPresent(hash.get128(), i);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -209,8 +209,13 @@ bool SLRUFileCachePriority::collectCandidatesForEvictionInProtected(
|
||||
{
|
||||
return false;
|
||||
}
|
||||
else
|
||||
chassert(downgrade_candidates->size() > 0);
|
||||
|
||||
/// We can have no downgrade candidates because cache size could
|
||||
/// reduce concurrently because of lock-free cache entries invalidation.
|
||||
if (downgrade_candidates->size() == 0)
|
||||
{
|
||||
return true;
|
||||
}
|
||||
|
||||
if (!probationary_queue.collectCandidatesForEviction(
|
||||
downgrade_stat.total_stat.releasable_size, downgrade_stat.total_stat.releasable_count,
|
||||
|
@ -53,8 +53,8 @@ void calculateMaxAndSum(Max & max, Sum & sum, T x)
|
||||
|
||||
ServerAsynchronousMetrics::ServerAsynchronousMetrics(
|
||||
ContextPtr global_context_,
|
||||
int update_period_seconds,
|
||||
int heavy_metrics_update_period_seconds,
|
||||
unsigned update_period_seconds,
|
||||
unsigned heavy_metrics_update_period_seconds,
|
||||
const ProtocolServerMetricsFunc & protocol_server_metrics_func_)
|
||||
: WithContext(global_context_)
|
||||
, AsynchronousMetrics(update_period_seconds, protocol_server_metrics_func_)
|
||||
|
@ -12,8 +12,8 @@ class ServerAsynchronousMetrics : WithContext, public AsynchronousMetrics
|
||||
public:
|
||||
ServerAsynchronousMetrics(
|
||||
ContextPtr global_context_,
|
||||
int update_period_seconds,
|
||||
int heavy_metrics_update_period_seconds,
|
||||
unsigned update_period_seconds,
|
||||
unsigned heavy_metrics_update_period_seconds,
|
||||
const ProtocolServerMetricsFunc & protocol_server_metrics_func_);
|
||||
~ServerAsynchronousMetrics() override;
|
||||
|
||||
|
@ -291,7 +291,7 @@ SystemLogs::SystemLogs(ContextPtr global_context, const Poco::Util::AbstractConf
|
||||
global_context, "system", "filesystem_read_prefetches_log", config, "filesystem_read_prefetches_log", "Contains a history of all prefetches done during reading from MergeTables backed by a remote filesystem.");
|
||||
asynchronous_metric_log = createSystemLog<AsynchronousMetricLog>(
|
||||
global_context, "system", "asynchronous_metric_log", config,
|
||||
"asynchronous_metric_log", "Contains the historical values for system.asynchronous_metrics, which are saved once per minute.");
|
||||
"asynchronous_metric_log", "Contains the historical values for system.asynchronous_metrics, once per time interval (one second by default).");
|
||||
opentelemetry_span_log = createSystemLog<OpenTelemetrySpanLog>(
|
||||
global_context, "system", "opentelemetry_span_log", config,
|
||||
"opentelemetry_span_log", "Contains information about trace spans for executed queries.");
|
||||
|
@ -1,4 +1,3 @@
|
||||
#include <Columns/Collator.h>
|
||||
#include <Parsers/ASTOrderByElement.h>
|
||||
#include <Common/SipHash.h>
|
||||
#include <IO/Operators.h>
|
||||
|
@ -601,6 +601,8 @@ public:
|
||||
|
||||
constexpr const char * getName() const override { return s.data(); }
|
||||
|
||||
Highlight highlight() const override { return Highlight::keyword; }
|
||||
|
||||
protected:
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override;
|
||||
};
|
||||
|
@ -278,7 +278,7 @@ bool ParserTableAsStringLiteralIdentifier::parseImpl(Pos & pos, ASTPtr & node, E
|
||||
bool ParserCompoundIdentifier::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
{
|
||||
ASTPtr id_list;
|
||||
if (!ParserList(std::make_unique<ParserIdentifier>(allow_query_parameter), std::make_unique<ParserToken>(TokenType::Dot), false)
|
||||
if (!ParserList(std::make_unique<ParserIdentifier>(allow_query_parameter, highlight_type), std::make_unique<ParserToken>(TokenType::Dot), false)
|
||||
.parse(pos, id_list, expected))
|
||||
return false;
|
||||
|
||||
@ -1491,7 +1491,7 @@ const char * ParserAlias::restricted_keywords[] =
|
||||
bool ParserAlias::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
{
|
||||
ParserKeyword s_as(Keyword::AS);
|
||||
ParserIdentifier id_p;
|
||||
ParserIdentifier id_p(false, Highlight::alias);
|
||||
|
||||
bool has_as_word = s_as.ignore(pos, expected);
|
||||
if (!allow_alias_without_as_keyword && !has_as_word)
|
||||
|
@ -25,12 +25,15 @@ protected:
|
||||
class ParserIdentifier : public IParserBase
|
||||
{
|
||||
public:
|
||||
explicit ParserIdentifier(bool allow_query_parameter_ = false) : allow_query_parameter(allow_query_parameter_) {}
|
||||
explicit ParserIdentifier(bool allow_query_parameter_ = false, Highlight highlight_type_ = Highlight::identifier)
|
||||
: allow_query_parameter(allow_query_parameter_), highlight_type(highlight_type_) {}
|
||||
Highlight highlight() const override { return highlight_type; }
|
||||
|
||||
protected:
|
||||
const char * getName() const override { return "identifier"; }
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override;
|
||||
bool allow_query_parameter;
|
||||
Highlight highlight_type;
|
||||
};
|
||||
|
||||
|
||||
@ -53,8 +56,8 @@ protected:
|
||||
class ParserCompoundIdentifier : public IParserBase
|
||||
{
|
||||
public:
|
||||
explicit ParserCompoundIdentifier(bool table_name_with_optional_uuid_ = false, bool allow_query_parameter_ = false)
|
||||
: table_name_with_optional_uuid(table_name_with_optional_uuid_), allow_query_parameter(allow_query_parameter_)
|
||||
explicit ParserCompoundIdentifier(bool table_name_with_optional_uuid_ = false, bool allow_query_parameter_ = false, Highlight highlight_type_ = Highlight::identifier)
|
||||
: table_name_with_optional_uuid(table_name_with_optional_uuid_), allow_query_parameter(allow_query_parameter_), highlight_type(highlight_type_)
|
||||
{
|
||||
}
|
||||
|
||||
@ -63,6 +66,7 @@ protected:
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override;
|
||||
bool table_name_with_optional_uuid;
|
||||
bool allow_query_parameter;
|
||||
Highlight highlight_type;
|
||||
};
|
||||
|
||||
/** *, t.*, db.table.*, COLUMNS('<regular expression>') APPLY(...) or EXCEPT(...) or REPLACE(...)
|
||||
@ -253,6 +257,7 @@ class ParserNumber : public IParserBase
|
||||
protected:
|
||||
const char * getName() const override { return "number"; }
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override;
|
||||
Highlight highlight() const override { return Highlight::number; }
|
||||
};
|
||||
|
||||
/** Unsigned integer, used in right hand side of tuple access operator (x.1).
|
||||
@ -273,6 +278,7 @@ class ParserStringLiteral : public IParserBase
|
||||
protected:
|
||||
const char * getName() const override { return "string literal"; }
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override;
|
||||
Highlight highlight() const override { return Highlight::string; }
|
||||
};
|
||||
|
||||
|
||||
@ -385,6 +391,7 @@ class ParserSubstitution : public IParserBase
|
||||
protected:
|
||||
const char * getName() const override { return "substitution"; }
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override;
|
||||
Highlight highlight() const override { return Highlight::substitution; }
|
||||
};
|
||||
|
||||
|
||||
|
@ -441,6 +441,21 @@ bool ParserKeyValuePairsList::parseImpl(Pos & pos, ASTPtr & node, Expected & exp
|
||||
return parser.parse(pos, node, expected);
|
||||
}
|
||||
|
||||
namespace
|
||||
{
|
||||
/// This wrapper is needed to highlight function names differently.
|
||||
class ParserFunctionName : public IParserBase
|
||||
{
|
||||
protected:
|
||||
const char * getName() const override { return "function name"; }
|
||||
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected) override
|
||||
{
|
||||
ParserCompoundIdentifier parser(false, true, Highlight::function);
|
||||
return parser.parse(pos, node, expected);
|
||||
}
|
||||
};
|
||||
}
|
||||
|
||||
|
||||
enum class Action
|
||||
{
|
||||
@ -809,6 +824,7 @@ struct ParserExpressionImpl
|
||||
|
||||
static const Operator finish_between_operator;
|
||||
|
||||
ParserFunctionName function_name_parser;
|
||||
ParserCompoundIdentifier identifier_parser{false, true};
|
||||
ParserNumber number_parser;
|
||||
ParserAsterisk asterisk_parser;
|
||||
@ -2359,7 +2375,7 @@ bool ParserFunction::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
{
|
||||
ASTPtr identifier;
|
||||
|
||||
if (ParserCompoundIdentifier(false,true).parse(pos, identifier, expected)
|
||||
if (ParserFunctionName().parse(pos, identifier, expected)
|
||||
&& ParserToken(TokenType::OpeningRoundBracket).ignore(pos, expected))
|
||||
{
|
||||
auto start = getFunctionLayer(identifier, is_table_function, allow_function_parameters);
|
||||
@ -2497,7 +2513,7 @@ Action ParserExpressionImpl::tryParseOperand(Layers & layers, IParser::Pos & pos
|
||||
{
|
||||
if (typeid_cast<ViewLayer *>(layers.back().get()) || typeid_cast<KustoLayer *>(layers.back().get()))
|
||||
{
|
||||
if (identifier_parser.parse(pos, tmp, expected)
|
||||
if (function_name_parser.parse(pos, tmp, expected)
|
||||
&& ParserToken(TokenType::OpeningRoundBracket).ignore(pos, expected))
|
||||
{
|
||||
layers.push_back(getFunctionLayer(tmp, layers.front()->is_table_function));
|
||||
@ -2629,50 +2645,53 @@ Action ParserExpressionImpl::tryParseOperand(Layers & layers, IParser::Pos & pos
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
}
|
||||
else if (identifier_parser.parse(pos, tmp, expected))
|
||||
else
|
||||
{
|
||||
if (pos->type == TokenType::OpeningRoundBracket)
|
||||
old_pos = pos;
|
||||
if (function_name_parser.parse(pos, tmp, expected) && pos->type == TokenType::OpeningRoundBracket)
|
||||
{
|
||||
++pos;
|
||||
layers.push_back(getFunctionLayer(tmp, layers.front()->is_table_function));
|
||||
return Action::OPERAND;
|
||||
}
|
||||
pos = old_pos;
|
||||
|
||||
if (identifier_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
}
|
||||
else if (substitution_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
}
|
||||
else if (pos->type == TokenType::OpeningRoundBracket)
|
||||
{
|
||||
|
||||
if (subquery_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
return Action::OPERATOR;
|
||||
}
|
||||
|
||||
++pos;
|
||||
layers.push_back(std::make_unique<RoundBracketsLayer>());
|
||||
return Action::OPERAND;
|
||||
}
|
||||
else if (pos->type == TokenType::OpeningSquareBracket)
|
||||
{
|
||||
++pos;
|
||||
layers.push_back(std::make_unique<ArrayLayer>());
|
||||
return Action::OPERAND;
|
||||
}
|
||||
else if (mysql_global_variable_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
}
|
||||
else
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
return Action::NONE;
|
||||
}
|
||||
}
|
||||
else if (substitution_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
}
|
||||
else if (pos->type == TokenType::OpeningRoundBracket)
|
||||
{
|
||||
|
||||
if (subquery_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
return Action::OPERATOR;
|
||||
}
|
||||
|
||||
++pos;
|
||||
layers.push_back(std::make_unique<RoundBracketsLayer>());
|
||||
return Action::OPERAND;
|
||||
}
|
||||
else if (pos->type == TokenType::OpeningSquareBracket)
|
||||
{
|
||||
++pos;
|
||||
layers.push_back(std::make_unique<ArrayLayer>());
|
||||
return Action::OPERAND;
|
||||
}
|
||||
else if (mysql_global_variable_parser.parse(pos, tmp, expected))
|
||||
{
|
||||
layers.back()->pushOperand(std::move(tmp));
|
||||
}
|
||||
else
|
||||
{
|
||||
return Action::NONE;
|
||||
}
|
||||
|
||||
return Action::OPERATOR;
|
||||
}
|
||||
|
@ -9,6 +9,7 @@ namespace ErrorCodes
|
||||
extern const int TOO_SLOW_PARSING;
|
||||
}
|
||||
|
||||
|
||||
IParser::Pos & IParser::Pos::operator=(const IParser::Pos & rhs)
|
||||
{
|
||||
depth = rhs.depth;
|
||||
@ -32,4 +33,29 @@ IParser::Pos & IParser::Pos::operator=(const IParser::Pos & rhs)
|
||||
return *this;
|
||||
}
|
||||
|
||||
|
||||
template <typename T>
|
||||
static bool intersects(T a_begin, T a_end, T b_begin, T b_end)
|
||||
{
|
||||
return (a_begin <= b_begin && b_begin < a_end)
|
||||
|| (b_begin <= a_begin && a_begin < b_end);
|
||||
}
|
||||
|
||||
|
||||
void Expected::highlight(HighlightedRange range)
|
||||
{
|
||||
if (!enable_highlighting)
|
||||
return;
|
||||
|
||||
auto it = highlights.lower_bound(range);
|
||||
while (it != highlights.end() && range.begin < it->end)
|
||||
{
|
||||
if (intersects(range.begin, range.end, it->begin, it->end))
|
||||
it = highlights.erase(it);
|
||||
else
|
||||
++it;
|
||||
}
|
||||
highlights.insert(range);
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -1,6 +1,7 @@
|
||||
#pragma once
|
||||
|
||||
#include <absl/container/inlined_vector.h>
|
||||
#include <set>
|
||||
#include <algorithm>
|
||||
#include <memory>
|
||||
|
||||
@ -21,14 +22,43 @@ namespace ErrorCodes
|
||||
extern const int LOGICAL_ERROR;
|
||||
}
|
||||
|
||||
enum class Highlight
|
||||
{
|
||||
none = 0,
|
||||
keyword,
|
||||
identifier,
|
||||
function,
|
||||
alias,
|
||||
substitution,
|
||||
number,
|
||||
string,
|
||||
};
|
||||
|
||||
struct HighlightedRange
|
||||
{
|
||||
const char * begin;
|
||||
const char * end;
|
||||
Highlight highlight;
|
||||
|
||||
auto operator<=>(const HighlightedRange & other) const
|
||||
{
|
||||
return begin <=> other.begin;
|
||||
}
|
||||
};
|
||||
|
||||
|
||||
/** Collects variants, how parser could proceed further at rightmost position.
|
||||
* Also collects a mapping of parsed ranges for highlighting,
|
||||
* which is accumulated through the parsing.
|
||||
*/
|
||||
struct Expected
|
||||
{
|
||||
absl::InlinedVector<const char *, 7> variants;
|
||||
const char * max_parsed_pos = nullptr;
|
||||
|
||||
bool enable_highlighting = false;
|
||||
std::set<HighlightedRange> highlights;
|
||||
|
||||
/// 'description' should be statically allocated string.
|
||||
ALWAYS_INLINE void add(const char * current_pos, const char * description)
|
||||
{
|
||||
@ -48,6 +78,8 @@ struct Expected
|
||||
{
|
||||
add(it->begin, description);
|
||||
}
|
||||
|
||||
void highlight(HighlightedRange range);
|
||||
};
|
||||
|
||||
|
||||
@ -158,6 +190,14 @@ public:
|
||||
return parse(pos, node, expected);
|
||||
}
|
||||
|
||||
/** If the parsed fragment should be highlighted in the query editor,
|
||||
* which type of highlighting to use?
|
||||
*/
|
||||
virtual Highlight highlight() const
|
||||
{
|
||||
return Highlight::none;
|
||||
}
|
||||
|
||||
virtual ~IParser() = default;
|
||||
};
|
||||
|
||||
|
@ -10,8 +10,25 @@ bool IParserBase::parse(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
|
||||
return wrapParseImpl(pos, IncreaseDepthTag{}, [&]
|
||||
{
|
||||
const char * begin = pos->begin;
|
||||
bool res = parseImpl(pos, node, expected);
|
||||
if (!res)
|
||||
if (res)
|
||||
{
|
||||
Highlight type = highlight();
|
||||
if (pos->begin > begin && type != Highlight::none)
|
||||
{
|
||||
Pos prev_token = pos;
|
||||
--prev_token;
|
||||
|
||||
HighlightedRange range;
|
||||
range.begin = begin;
|
||||
range.end = prev_token->end;
|
||||
range.highlight = type;
|
||||
|
||||
expected.highlight(range);
|
||||
}
|
||||
}
|
||||
else
|
||||
node = nullptr;
|
||||
return res;
|
||||
});
|
||||
|
@ -40,7 +40,6 @@ bool ParserInsertQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
ParserKeyword s_with(Keyword::WITH);
|
||||
ParserToken s_lparen(TokenType::OpeningRoundBracket);
|
||||
ParserToken s_rparen(TokenType::ClosingRoundBracket);
|
||||
ParserToken s_semicolon(TokenType::Semicolon);
|
||||
ParserIdentifier name_p(true);
|
||||
ParserList columns_p(std::make_unique<ParserInsertElement>(), std::make_unique<ParserToken>(TokenType::Comma), false);
|
||||
ParserFunction table_function_p{false};
|
||||
@ -147,8 +146,9 @@ bool ParserInsertQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
|
||||
{
|
||||
/// If VALUES is defined in query, everything except setting will be parsed as data,
|
||||
/// and if values followed by semicolon, the data should be null.
|
||||
if (!s_semicolon.checkWithoutMoving(pos, expected))
|
||||
if (pos->type != TokenType::Semicolon)
|
||||
data = pos->begin;
|
||||
|
||||
format_str = "Values";
|
||||
}
|
||||
else if (s_format.ignore(pos, expected))
|
||||
|
@ -60,21 +60,6 @@ bool parseDatabaseAndTableAsAST(IParser::Pos & pos, Expected & expected, ASTPtr
|
||||
}
|
||||
|
||||
|
||||
bool parseDatabase(IParser::Pos & pos, Expected & expected, String & database_str)
|
||||
{
|
||||
ParserToken s_dot(TokenType::Dot);
|
||||
ParserIdentifier identifier_parser;
|
||||
|
||||
ASTPtr database;
|
||||
database_str = "";
|
||||
|
||||
if (!identifier_parser.parse(pos, database, expected))
|
||||
return false;
|
||||
|
||||
tryGetIdentifierNameInto(database, database_str);
|
||||
return true;
|
||||
}
|
||||
|
||||
bool parseDatabaseAsAST(IParser::Pos & pos, Expected & expected, ASTPtr & database)
|
||||
{
|
||||
ParserIdentifier identifier_parser(/* allow_query_parameter */true);
|
||||
|
@ -226,6 +226,32 @@ std::string getUnmatchedParenthesesErrorMessage(
|
||||
}
|
||||
|
||||
|
||||
static ASTInsertQuery * getInsertAST(const ASTPtr & ast)
|
||||
{
|
||||
/// Either it is INSERT or EXPLAIN INSERT.
|
||||
if (auto * explain = ast->as<ASTExplainQuery>())
|
||||
{
|
||||
if (auto explained_query = explain->getExplainedQuery())
|
||||
{
|
||||
return explained_query->as<ASTInsertQuery>();
|
||||
}
|
||||
}
|
||||
else
|
||||
{
|
||||
return ast->as<ASTInsertQuery>();
|
||||
}
|
||||
|
||||
return nullptr;
|
||||
}
|
||||
|
||||
const char * getInsertData(const ASTPtr & ast)
|
||||
{
|
||||
if (const ASTInsertQuery * insert = getInsertAST(ast))
|
||||
return insert->data;
|
||||
return nullptr;
|
||||
}
|
||||
|
||||
|
||||
ASTPtr tryParseQuery(
|
||||
IParser & parser,
|
||||
const char * & _out_query_end, /* also query begin as input parameter */
|
||||
@ -270,29 +296,11 @@ ASTPtr tryParseQuery(
|
||||
if (res && max_parser_depth)
|
||||
res->checkDepth(max_parser_depth);
|
||||
|
||||
ASTInsertQuery * insert = nullptr;
|
||||
if (parse_res)
|
||||
{
|
||||
if (auto * explain = res->as<ASTExplainQuery>())
|
||||
{
|
||||
if (auto explained_query = explain->getExplainedQuery())
|
||||
{
|
||||
insert = explained_query->as<ASTInsertQuery>();
|
||||
}
|
||||
}
|
||||
else
|
||||
{
|
||||
insert = res->as<ASTInsertQuery>();
|
||||
}
|
||||
}
|
||||
|
||||
// If parsed query ends at data for insertion. Data for insertion could be
|
||||
// in any format and not necessary be lexical correct, so we can't perform
|
||||
// most of the checks.
|
||||
if (insert && insert->data)
|
||||
{
|
||||
/// If parsed query ends at data for insertion. Data for insertion could be
|
||||
/// in any format and not necessary be lexical correct, so we can't perform
|
||||
/// most of the checks.
|
||||
if (res && getInsertData(res))
|
||||
return res;
|
||||
}
|
||||
|
||||
// More granular checks for queries other than INSERT w/inline data.
|
||||
/// Lexical error
|
||||
@ -434,11 +442,9 @@ std::pair<const char *, bool> splitMultipartQuery(
|
||||
|
||||
ast = parseQueryAndMovePosition(parser, pos, end, "", true, max_query_size, max_parser_depth, max_parser_backtracks);
|
||||
|
||||
auto * insert = ast->as<ASTInsertQuery>();
|
||||
|
||||
if (insert && insert->data)
|
||||
if (ASTInsertQuery * insert = getInsertAST(ast))
|
||||
{
|
||||
/// Data for INSERT is broken on new line
|
||||
/// Data for INSERT is broken on the new line
|
||||
pos = insert->data;
|
||||
while (*pos && *pos != '\n')
|
||||
++pos;
|
||||
|
@ -71,4 +71,9 @@ std::pair<const char *, bool> splitMultipartQuery(
|
||||
size_t max_parser_backtracks,
|
||||
bool allow_settings_after_format_in_insert);
|
||||
|
||||
/** If the query contains raw data part, such as INSERT ... FORMAT ..., return a pointer to it.
|
||||
* The SQL parser stops at the raw data part, which is parsed by a separate parser.
|
||||
*/
|
||||
const char * getInsertData(const ASTPtr & ast);
|
||||
|
||||
}
|
||||
|
@ -86,7 +86,7 @@ size_t ProtobufListInputFormat::countRows(size_t max_block_size)
|
||||
ProtobufListSchemaReader::ProtobufListSchemaReader(const FormatSettings & format_settings)
|
||||
: schema_info(
|
||||
format_settings.schema.format_schema, "Protobuf", true, format_settings.schema.is_server, format_settings.schema.format_schema_path)
|
||||
, skip_unsopported_fields(format_settings.protobuf.skip_fields_with_unsupported_types_in_schema_inference)
|
||||
, skip_unsupported_fields(format_settings.protobuf.skip_fields_with_unsupported_types_in_schema_inference)
|
||||
, google_protos_path(format_settings.protobuf.google_protos_path)
|
||||
{
|
||||
}
|
||||
@ -95,7 +95,7 @@ NamesAndTypesList ProtobufListSchemaReader::readSchema()
|
||||
{
|
||||
const auto * message_descriptor
|
||||
= ProtobufSchemas::instance().getMessageTypeForFormatSchema(schema_info, ProtobufSchemas::WithEnvelope::Yes, google_protos_path);
|
||||
return protobufSchemaToCHSchema(message_descriptor, skip_unsopported_fields);
|
||||
return protobufSchemaToCHSchema(message_descriptor, skip_unsupported_fields);
|
||||
}
|
||||
|
||||
void registerInputFormatProtobufList(FormatFactory & factory)
|
||||
|
@ -56,7 +56,7 @@ public:
|
||||
|
||||
private:
|
||||
const FormatSchemaInfo schema_info;
|
||||
bool skip_unsopported_fields;
|
||||
bool skip_unsupported_fields;
|
||||
const String google_protos_path;
|
||||
};
|
||||
|
||||
|
@ -432,13 +432,12 @@ AggregateProjectionCandidates getAggregateProjectionCandidates(
|
||||
{
|
||||
const auto & keys = aggregating.getParams().keys;
|
||||
const auto & aggregates = aggregating.getParams().aggregates;
|
||||
Block key_virtual_columns = reading.getMergeTreeData().getHeaderWithVirtualsForFilter();
|
||||
const auto metadata = reading.getStorageMetadata();
|
||||
Block key_virtual_columns = reading.getMergeTreeData().getHeaderWithVirtualsForFilter(metadata);
|
||||
|
||||
AggregateProjectionCandidates candidates;
|
||||
|
||||
const auto & parts = reading.getParts();
|
||||
|
||||
const auto metadata = reading.getStorageMetadata();
|
||||
ContextPtr context = reading.getContext();
|
||||
|
||||
const auto & projections = metadata->projections;
|
||||
|
@ -1415,7 +1415,8 @@ static void buildIndexes(
|
||||
indexes->partition_pruner.emplace(metadata_snapshot, filter_actions_dag, context, false /* strict */);
|
||||
}
|
||||
|
||||
indexes->part_values = MergeTreeDataSelectExecutor::filterPartsByVirtualColumns(data, parts, filter_actions_dag, context);
|
||||
indexes->part_values
|
||||
= MergeTreeDataSelectExecutor::filterPartsByVirtualColumns(metadata_snapshot, data, parts, filter_actions_dag, context);
|
||||
MergeTreeDataSelectExecutor::buildKeyConditionFromPartOffset(indexes->part_offset_condition, filter_actions_dag, context);
|
||||
|
||||
indexes->use_skip_indexes = settings.use_skip_indexes;
|
||||
|
@ -1031,19 +1031,26 @@ void MergeTreeData::MergingParams::check(const StorageInMemoryMetadata & metadat
|
||||
|
||||
const Names MergeTreeData::virtuals_useful_for_filter = {"_part", "_partition_id", "_part_uuid", "_partition_value", "_part_data_version"};
|
||||
|
||||
Block MergeTreeData::getHeaderWithVirtualsForFilter() const
|
||||
Block MergeTreeData::getHeaderWithVirtualsForFilter(const StorageMetadataPtr & metadata) const
|
||||
{
|
||||
const auto columns = metadata->getColumns().getAllPhysical();
|
||||
Block header;
|
||||
auto virtuals_desc = getVirtualsPtr();
|
||||
for (const auto & name : virtuals_useful_for_filter)
|
||||
{
|
||||
if (columns.contains(name))
|
||||
continue;
|
||||
if (auto column = virtuals_desc->tryGet(name))
|
||||
header.insert({column->type->createColumn(), column->type, name});
|
||||
}
|
||||
|
||||
return header;
|
||||
}
|
||||
|
||||
Block MergeTreeData::getBlockWithVirtualsForFilter(const MergeTreeData::DataPartsVector & parts, bool ignore_empty) const
|
||||
Block MergeTreeData::getBlockWithVirtualsForFilter(
|
||||
const StorageMetadataPtr & metadata, const MergeTreeData::DataPartsVector & parts, bool ignore_empty) const
|
||||
{
|
||||
auto block = getHeaderWithVirtualsForFilter();
|
||||
auto block = getHeaderWithVirtualsForFilter(metadata);
|
||||
|
||||
for (const auto & part_or_projection : parts)
|
||||
{
|
||||
@ -1072,7 +1079,7 @@ std::optional<UInt64> MergeTreeData::totalRowsByPartitionPredicateImpl(
|
||||
return 0;
|
||||
|
||||
auto metadata_snapshot = getInMemoryMetadataPtr();
|
||||
auto virtual_columns_block = getBlockWithVirtualsForFilter({parts[0]});
|
||||
auto virtual_columns_block = getBlockWithVirtualsForFilter(metadata_snapshot, {parts[0]});
|
||||
|
||||
auto filter_dag = VirtualColumnUtils::splitFilterDagForAllowedInputs(filter_actions_dag->getOutputs().at(0), nullptr);
|
||||
if (!filter_dag)
|
||||
@ -1091,7 +1098,7 @@ std::optional<UInt64> MergeTreeData::totalRowsByPartitionPredicateImpl(
|
||||
std::unordered_set<String> part_values;
|
||||
if (valid)
|
||||
{
|
||||
virtual_columns_block = getBlockWithVirtualsForFilter(parts);
|
||||
virtual_columns_block = getBlockWithVirtualsForFilter(metadata_snapshot, parts);
|
||||
VirtualColumnUtils::filterBlockWithDAG(filter_dag, virtual_columns_block, local_context);
|
||||
part_values = VirtualColumnUtils::extractSingleValueFromBlock<String>(virtual_columns_block, "_part");
|
||||
if (part_values.empty())
|
||||
@ -6694,11 +6701,11 @@ Block MergeTreeData::getMinMaxCountProjectionBlock(
|
||||
};
|
||||
|
||||
Block virtual_columns_block;
|
||||
auto virtual_block = getHeaderWithVirtualsForFilter();
|
||||
auto virtual_block = getHeaderWithVirtualsForFilter(metadata_snapshot);
|
||||
bool has_virtual_column = std::any_of(required_columns.begin(), required_columns.end(), [&](const auto & name) { return virtual_block.has(name); });
|
||||
if (has_virtual_column || filter_dag)
|
||||
{
|
||||
virtual_columns_block = getBlockWithVirtualsForFilter(parts, /*ignore_empty=*/ true);
|
||||
virtual_columns_block = getBlockWithVirtualsForFilter(metadata_snapshot, parts, /*ignore_empty=*/true);
|
||||
if (virtual_columns_block.rows() == 0)
|
||||
return {};
|
||||
}
|
||||
@ -7074,7 +7081,7 @@ MergeTreeData & MergeTreeData::checkStructureAndGetMergeTreeData(
|
||||
return checkStructureAndGetMergeTreeData(*source_table, src_snapshot, my_snapshot);
|
||||
}
|
||||
|
||||
std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> MergeTreeData::cloneAndLoadDataPart(
|
||||
std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> MergeTreeData::cloneAndLoadDataPartOnSameDisk(
|
||||
const MergeTreeData::DataPartPtr & src_part,
|
||||
const String & tmp_part_prefix,
|
||||
const MergeTreePartInfo & dst_part_info,
|
||||
@ -7084,23 +7091,28 @@ std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> MergeTreeData::cloneAn
|
||||
const WriteSettings & write_settings)
|
||||
{
|
||||
chassert(!isStaticStorage());
|
||||
bool on_same_disk = false;
|
||||
for (const DiskPtr & disk : this->getStoragePolicy()->getDisks())
|
||||
|
||||
/// Check that the storage policy contains the disk where the src_part is located.
|
||||
bool does_storage_policy_allow_same_disk = false;
|
||||
for (const DiskPtr & disk : getStoragePolicy()->getDisks())
|
||||
{
|
||||
if (disk->getName() == src_part->getDataPartStorage().getDiskName())
|
||||
{
|
||||
on_same_disk = true;
|
||||
does_storage_policy_allow_same_disk = true;
|
||||
break;
|
||||
}
|
||||
}
|
||||
|
||||
if (!does_storage_policy_allow_same_disk)
|
||||
throw Exception(
|
||||
ErrorCodes::BAD_ARGUMENTS,
|
||||
"Could not clone and load part {} because disk does not belong to storage policy",
|
||||
quoteString(src_part->getDataPartStorage().getFullPath()));
|
||||
|
||||
String dst_part_name = src_part->getNewName(dst_part_info);
|
||||
String tmp_dst_part_name = tmp_part_prefix + dst_part_name;
|
||||
auto temporary_directory_lock = getTemporaryPartDirectoryHolder(tmp_dst_part_name);
|
||||
|
||||
/// Why it is needed if we only hardlink files?
|
||||
/// Answer: In issue #59377, add copy when attach from different disk.
|
||||
auto reservation = src_part->getDataPartStorage().reserve(src_part->getBytesOnDisk());
|
||||
auto src_part_storage = src_part->getDataPartStoragePtr();
|
||||
|
||||
@ -7108,30 +7120,16 @@ std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> MergeTreeData::cloneAn
|
||||
MergeTreeData::MutableDataPartPtr src_flushed_tmp_part;
|
||||
|
||||
String with_copy;
|
||||
if (params.copy_instead_of_hardlink || !on_same_disk)
|
||||
if (params.copy_instead_of_hardlink)
|
||||
with_copy = " (copying data)";
|
||||
|
||||
|
||||
std::shared_ptr<IDataPartStorage> dst_part_storage{};
|
||||
if (on_same_disk && !params.copy_instead_of_hardlink)
|
||||
{
|
||||
dst_part_storage = src_part_storage->freeze(
|
||||
relative_data_path,
|
||||
tmp_dst_part_name,
|
||||
read_settings,
|
||||
write_settings,
|
||||
/* save_metadata_callback= */ {},
|
||||
params);
|
||||
}
|
||||
else
|
||||
{
|
||||
auto reservation_on_dst = getStoragePolicy()->reserve(src_part->getBytesOnDisk());
|
||||
if (!reservation_on_dst)
|
||||
throw Exception(ErrorCodes::NOT_ENOUGH_SPACE, "Not enough space on disk.");
|
||||
dst_part_storage = src_part_storage->clonePart(
|
||||
this->getRelativeDataPath(), tmp_dst_part_name, reservation_on_dst->getDisk(), read_settings, write_settings, {}, {});
|
||||
}
|
||||
|
||||
auto dst_part_storage = src_part_storage->freeze(
|
||||
relative_data_path,
|
||||
tmp_dst_part_name,
|
||||
read_settings,
|
||||
write_settings,
|
||||
/* save_metadata_callback= */ {},
|
||||
params);
|
||||
|
||||
if (params.metadata_version_to_write.has_value())
|
||||
{
|
||||
@ -7153,7 +7151,7 @@ std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> MergeTreeData::cloneAn
|
||||
.withPartFormatFromDisk()
|
||||
.build();
|
||||
|
||||
if (on_same_disk && !params.copy_instead_of_hardlink && params.hardlinked_files)
|
||||
if (!params.copy_instead_of_hardlink && params.hardlinked_files)
|
||||
{
|
||||
params.hardlinked_files->source_part_name = src_part->name;
|
||||
params.hardlinked_files->source_table_shared_id = src_part->storage.getTableSharedID();
|
||||
@ -7197,7 +7195,6 @@ std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> MergeTreeData::cloneAn
|
||||
return std::make_pair(dst_data_part, std::move(temporary_directory_lock));
|
||||
}
|
||||
|
||||
|
||||
String MergeTreeData::getFullPathOnDisk(const DiskPtr & disk) const
|
||||
{
|
||||
return disk->getPath() + relative_data_path;
|
||||
|
@ -839,7 +839,7 @@ public:
|
||||
MergeTreeData & checkStructureAndGetMergeTreeData(const StoragePtr & source_table, const StorageMetadataPtr & src_snapshot, const StorageMetadataPtr & my_snapshot) const;
|
||||
MergeTreeData & checkStructureAndGetMergeTreeData(IStorage & source_table, const StorageMetadataPtr & src_snapshot, const StorageMetadataPtr & my_snapshot) const;
|
||||
|
||||
std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> cloneAndLoadDataPart(
|
||||
std::pair<MergeTreeData::MutableDataPartPtr, scope_guard> cloneAndLoadDataPartOnSameDisk(
|
||||
const MergeTreeData::DataPartPtr & src_part,
|
||||
const String & tmp_part_prefix,
|
||||
const MergeTreePartInfo & dst_part_info,
|
||||
@ -990,10 +990,11 @@ public:
|
||||
static const Names virtuals_useful_for_filter;
|
||||
|
||||
/// Construct a sample block of virtual columns.
|
||||
Block getHeaderWithVirtualsForFilter() const;
|
||||
Block getHeaderWithVirtualsForFilter(const StorageMetadataPtr & metadata) const;
|
||||
|
||||
/// Construct a block consisting only of possible virtual columns for part pruning.
|
||||
Block getBlockWithVirtualsForFilter(const MergeTreeData::DataPartsVector & parts, bool ignore_empty = false) const;
|
||||
Block getBlockWithVirtualsForFilter(
|
||||
const StorageMetadataPtr & metadata, const MergeTreeData::DataPartsVector & parts, bool ignore_empty = false) const;
|
||||
|
||||
/// In merge tree we do inserts with several steps. One of them:
|
||||
/// X. write part to temporary directory with some temp name
|
||||
|
@ -473,6 +473,7 @@ void MergeTreeDataSelectExecutor::buildKeyConditionFromPartOffset(
|
||||
}
|
||||
|
||||
std::optional<std::unordered_set<String>> MergeTreeDataSelectExecutor::filterPartsByVirtualColumns(
|
||||
const StorageMetadataPtr & metadata_snapshot,
|
||||
const MergeTreeData & data,
|
||||
const MergeTreeData::DataPartsVector & parts,
|
||||
const ActionsDAGPtr & filter_dag,
|
||||
@ -481,12 +482,12 @@ std::optional<std::unordered_set<String>> MergeTreeDataSelectExecutor::filterPar
|
||||
if (!filter_dag)
|
||||
return {};
|
||||
|
||||
auto sample = data.getHeaderWithVirtualsForFilter();
|
||||
auto sample = data.getHeaderWithVirtualsForFilter(metadata_snapshot);
|
||||
auto dag = VirtualColumnUtils::splitFilterDagForAllowedInputs(filter_dag->getOutputs().at(0), &sample);
|
||||
if (!dag)
|
||||
return {};
|
||||
|
||||
auto virtual_columns_block = data.getBlockWithVirtualsForFilter(parts);
|
||||
auto virtual_columns_block = data.getBlockWithVirtualsForFilter(metadata_snapshot, parts);
|
||||
VirtualColumnUtils::filterBlockWithDAG(dag, virtual_columns_block, context);
|
||||
return VirtualColumnUtils::extractSingleValueFromBlock<String>(virtual_columns_block, "_part");
|
||||
}
|
||||
|
@ -166,6 +166,7 @@ public:
|
||||
/// Example: SELECT count() FROM table WHERE _part = 'part_name'
|
||||
/// If expression found, return a set with allowed part names (std::nullopt otherwise).
|
||||
static std::optional<std::unordered_set<String>> filterPartsByVirtualColumns(
|
||||
const StorageMetadataPtr & metadata_snapshot,
|
||||
const MergeTreeData & data,
|
||||
const MergeTreeData::DataPartsVector & parts,
|
||||
const ActionsDAGPtr & filter_dag,
|
||||
|
@ -2146,7 +2146,7 @@ bool MutateTask::prepare()
|
||||
scope_guard lock;
|
||||
|
||||
{
|
||||
std::tie(part, lock) = ctx->data->cloneAndLoadDataPart(
|
||||
std::tie(part, lock) = ctx->data->cloneAndLoadDataPartOnSameDisk(
|
||||
ctx->source_part, prefix, ctx->future_part->part_info, ctx->metadata_snapshot, clone_params, ctx->context->getReadSettings(), ctx->context->getWriteSettings());
|
||||
part->getDataPartStorage().beginTransaction();
|
||||
ctx->temporary_directory_lock = std::move(lock);
|
||||
|
@ -2096,7 +2096,7 @@ void StorageMergeTree::replacePartitionFrom(const StoragePtr & source_table, con
|
||||
MergeTreePartInfo dst_part_info(partition_id, temp_index, temp_index, src_part->info.level);
|
||||
|
||||
IDataPartStorage::ClonePartParams clone_params{.txn = local_context->getCurrentTransaction()};
|
||||
auto [dst_part, part_lock] = cloneAndLoadDataPart(
|
||||
auto [dst_part, part_lock] = cloneAndLoadDataPartOnSameDisk(
|
||||
src_part,
|
||||
TMP_PREFIX,
|
||||
dst_part_info,
|
||||
@ -2207,7 +2207,7 @@ void StorageMergeTree::movePartitionToTable(const StoragePtr & dest_table, const
|
||||
.copy_instead_of_hardlink = getSettings()->always_use_copy_instead_of_hardlinks,
|
||||
};
|
||||
|
||||
auto [dst_part, part_lock] = dest_table_storage->cloneAndLoadDataPart(
|
||||
auto [dst_part, part_lock] = dest_table_storage->cloneAndLoadDataPartOnSameDisk(
|
||||
src_part,
|
||||
TMP_PREFIX,
|
||||
dst_part_info,
|
||||
|
@ -2788,7 +2788,7 @@ bool StorageReplicatedMergeTree::executeReplaceRange(LogEntry & entry)
|
||||
|
||||
auto obtain_part = [&] (PartDescriptionPtr & part_desc)
|
||||
{
|
||||
/// Fetches with zero-copy-replication are cheap, but cloneAndLoadDataPart(OnSameDisk) will do full copy.
|
||||
/// Fetches with zero-copy-replication are cheap, but cloneAndLoadDataPartOnSameDisk will do full copy.
|
||||
/// It's okay to check the setting for current table and disk for the source table, because src and dst part are on the same disk.
|
||||
bool prefer_fetch_from_other_replica = !part_desc->replica.empty() && storage_settings_ptr->allow_remote_fs_zero_copy_replication
|
||||
&& part_desc->src_table_part && part_desc->src_table_part->isStoredOnRemoteDiskWithZeroCopySupport();
|
||||
@ -2807,7 +2807,7 @@ bool StorageReplicatedMergeTree::executeReplaceRange(LogEntry & entry)
|
||||
.copy_instead_of_hardlink = storage_settings_ptr->always_use_copy_instead_of_hardlinks || ((our_zero_copy_enabled || source_zero_copy_enabled) && part_desc->src_table_part->isStoredOnRemoteDiskWithZeroCopySupport()),
|
||||
.metadata_version_to_write = metadata_snapshot->getMetadataVersion()
|
||||
};
|
||||
auto [res_part, temporary_part_lock] = cloneAndLoadDataPart(
|
||||
auto [res_part, temporary_part_lock] = cloneAndLoadDataPartOnSameDisk(
|
||||
part_desc->src_table_part,
|
||||
TMP_PREFIX + "clone_",
|
||||
part_desc->new_part_info,
|
||||
@ -4888,7 +4888,7 @@ bool StorageReplicatedMergeTree::fetchPart(
|
||||
.keep_metadata_version = true,
|
||||
};
|
||||
|
||||
auto [cloned_part, lock] = cloneAndLoadDataPart(
|
||||
auto [cloned_part, lock] = cloneAndLoadDataPartOnSameDisk(
|
||||
part_to_clone,
|
||||
"tmp_clone_",
|
||||
part_info,
|
||||
@ -8078,14 +8078,12 @@ void StorageReplicatedMergeTree::replacePartitionFrom(
|
||||
|
||||
bool zero_copy_enabled = storage_settings_ptr->allow_remote_fs_zero_copy_replication
|
||||
|| dynamic_cast<const MergeTreeData *>(source_table.get())->getSettings()->allow_remote_fs_zero_copy_replication;
|
||||
|
||||
IDataPartStorage::ClonePartParams clone_params
|
||||
{
|
||||
.copy_instead_of_hardlink = storage_settings_ptr->always_use_copy_instead_of_hardlinks || (zero_copy_enabled && src_part->isStoredOnRemoteDiskWithZeroCopySupport()),
|
||||
.metadata_version_to_write = metadata_snapshot->getMetadataVersion()
|
||||
};
|
||||
|
||||
auto [dst_part, part_lock] = cloneAndLoadDataPart(
|
||||
auto [dst_part, part_lock] = cloneAndLoadDataPartOnSameDisk(
|
||||
src_part,
|
||||
TMP_PREFIX,
|
||||
dst_part_info,
|
||||
@ -8093,10 +8091,9 @@ void StorageReplicatedMergeTree::replacePartitionFrom(
|
||||
clone_params,
|
||||
query_context->getReadSettings(),
|
||||
query_context->getWriteSettings());
|
||||
|
||||
dst_parts.emplace_back(std::move(dst_part));
|
||||
dst_parts_locks.emplace_back(std::move(part_lock));
|
||||
src_parts.emplace_back(src_part);
|
||||
dst_parts.emplace_back(dst_part);
|
||||
dst_parts_locks.emplace_back(std::move(part_lock));
|
||||
ephemeral_locks.emplace_back(std::move(*lock));
|
||||
block_id_paths.emplace_back(block_id_path);
|
||||
part_checksums.emplace_back(hash_hex);
|
||||
@ -8349,7 +8346,7 @@ void StorageReplicatedMergeTree::movePartitionToTable(const StoragePtr & dest_ta
|
||||
.copy_instead_of_hardlink = storage_settings_ptr->always_use_copy_instead_of_hardlinks || (zero_copy_enabled && src_part->isStoredOnRemoteDiskWithZeroCopySupport()),
|
||||
.metadata_version_to_write = dest_metadata_snapshot->getMetadataVersion()
|
||||
};
|
||||
auto [dst_part, dst_part_lock] = dest_table_storage->cloneAndLoadDataPart(
|
||||
auto [dst_part, dst_part_lock] = dest_table_storage->cloneAndLoadDataPartOnSameDisk(
|
||||
src_part,
|
||||
TMP_PREFIX,
|
||||
dst_part_info,
|
||||
|
@ -1436,7 +1436,7 @@ def _configure_jobs(
|
||||
)
|
||||
|
||||
if pr_info.is_merge_queue():
|
||||
# no need to run pending job in MQ, since it's pending - it's not affected by current checnge
|
||||
# do not run in-progress jobs in MQ, since it's in-progress current change does not affect it (it's digest)
|
||||
for job_to_wait in jobs_to_wait:
|
||||
if job_to_wait in jobs_to_do:
|
||||
print(f"Remove pending job [{job_to_wait}] from MQ workflow")
|
||||
|
@ -9,7 +9,7 @@ from get_robot_token import get_best_robot_token
|
||||
from pr_info import PRInfo
|
||||
from github_helper import GitHub
|
||||
from commit_status_helper import get_commit, post_commit_status
|
||||
from report import FAILURE, SUCCESS
|
||||
from report import SUCCESS
|
||||
|
||||
|
||||
def parse_args() -> argparse.Namespace:
|
||||
@ -53,14 +53,38 @@ def merge_sync_pr(gh, sync_pr):
|
||||
|
||||
|
||||
def set_sync_status(gh, pr_info, sync_pr):
|
||||
if not sync_pr or not sync_pr.mergeable:
|
||||
print("Sync PR is not mergeable")
|
||||
post_commit_status(
|
||||
get_commit(gh, pr_info.sha), FAILURE, "", "Sync PR failure", "A Sync"
|
||||
)
|
||||
else:
|
||||
print("Sync PR is mergeable")
|
||||
# FIXME: uncomment posting red Sync status to prohibit merge in MQ if PR state fetching works good
|
||||
if not sync_pr:
|
||||
# post_commit_status(
|
||||
# get_commit(gh, pr_info.sha), FAILURE, "", "Sync PR not found", "A Sync"
|
||||
# )
|
||||
return
|
||||
|
||||
# FIXME: fetch sync pr in a proper way
|
||||
# retries = 0
|
||||
# while sync_pr.mergeable_state == "unknown" and retries < 3:
|
||||
# retries += 1
|
||||
# print(f"Unknown status. Trying to fetch again [{retries}/3]")
|
||||
# time.sleep(5)
|
||||
# sync_pr = gh.get_pulls_from_search(
|
||||
# query=f"head:sync-upstream/pr/{sync_pr.number} org:ClickHouse type:pr",
|
||||
# repo="ClickHouse/clickhouse-private",
|
||||
# )
|
||||
|
||||
if sync_pr.mergeable_state == "clean":
|
||||
print(f"Sync PR [{sync_pr.number}] is clean")
|
||||
post_commit_status(get_commit(gh, pr_info.sha), SUCCESS, "", "", "A Sync")
|
||||
else:
|
||||
print(
|
||||
f"Sync PR [{sync_pr}] is not mergeable, state [{sync_pr.mergeable_state}]"
|
||||
)
|
||||
# post_commit_status(
|
||||
# get_commit(gh, pr_info.sha),
|
||||
# FAILURE,
|
||||
# "",
|
||||
# f"state: {sync_pr.mergeable_state}",
|
||||
# "A Sync",
|
||||
# )
|
||||
|
||||
|
||||
def main():
|
||||
|
@ -1,17 +0,0 @@
|
||||
<clickhouse>
|
||||
<remote_servers>
|
||||
<test_cluster>
|
||||
<shard>
|
||||
<internal_replication>true</internal_replication>
|
||||
<replica>
|
||||
<host>replica1</host>
|
||||
<port>9000</port>
|
||||
</replica>
|
||||
<replica>
|
||||
<host>replica2</host>
|
||||
<port>9000</port>
|
||||
</replica>
|
||||
</shard>
|
||||
</test_cluster>
|
||||
</remote_servers>
|
||||
</clickhouse>
|
@ -1,187 +0,0 @@
|
||||
import pytest
|
||||
from helpers.cluster import ClickHouseCluster
|
||||
from helpers.test_tools import assert_eq_with_retry
|
||||
|
||||
cluster = ClickHouseCluster(__file__)
|
||||
|
||||
replica1 = cluster.add_instance(
|
||||
"replica1", with_zookeeper=True, main_configs=["configs/remote_servers.xml"]
|
||||
)
|
||||
replica2 = cluster.add_instance(
|
||||
"replica2", with_zookeeper=True, main_configs=["configs/remote_servers.xml"]
|
||||
)
|
||||
|
||||
|
||||
@pytest.fixture(scope="module")
|
||||
def start_cluster():
|
||||
try:
|
||||
cluster.start()
|
||||
yield cluster
|
||||
except Exception as ex:
|
||||
print(ex)
|
||||
finally:
|
||||
cluster.shutdown()
|
||||
|
||||
|
||||
def cleanup(nodes):
|
||||
for node in nodes:
|
||||
node.query("DROP TABLE IF EXISTS source SYNC")
|
||||
node.query("DROP TABLE IF EXISTS destination SYNC")
|
||||
|
||||
|
||||
def create_source_table(node, table_name, replicated):
|
||||
replica = node.name
|
||||
engine = (
|
||||
f"ReplicatedMergeTree('/clickhouse/tables/1/{table_name}', '{replica}')"
|
||||
if replicated
|
||||
else "MergeTree()"
|
||||
)
|
||||
node.query_with_retry(
|
||||
"""
|
||||
ATTACH TABLE {table_name} UUID 'cf712b4f-2ca8-435c-ac23-c4393efe52f7'
|
||||
(
|
||||
price UInt32,
|
||||
date Date,
|
||||
postcode1 LowCardinality(String),
|
||||
postcode2 LowCardinality(String),
|
||||
type Enum8('other' = 0, 'terraced' = 1, 'semi-detached' = 2, 'detached' = 3, 'flat' = 4),
|
||||
is_new UInt8,
|
||||
duration Enum8('unknown' = 0, 'freehold' = 1, 'leasehold' = 2),
|
||||
addr1 String,
|
||||
addr2 String,
|
||||
street LowCardinality(String),
|
||||
locality LowCardinality(String),
|
||||
town LowCardinality(String),
|
||||
district LowCardinality(String),
|
||||
county LowCardinality(String)
|
||||
)
|
||||
ENGINE = {engine}
|
||||
ORDER BY (postcode1, postcode2, addr1, addr2)
|
||||
SETTINGS disk = disk(type = web, endpoint = 'https://raw.githubusercontent.com/ClickHouse/web-tables-demo/main/web/')
|
||||
""".format(
|
||||
table_name=table_name, engine=engine
|
||||
)
|
||||
)
|
||||
|
||||
|
||||
def create_destination_table(node, table_name, replicated):
|
||||
replica = node.name
|
||||
engine = (
|
||||
f"ReplicatedMergeTree('/clickhouse/tables/1/{table_name}', '{replica}')"
|
||||
if replicated
|
||||
else "MergeTree()"
|
||||
)
|
||||
node.query_with_retry(
|
||||
"""
|
||||
CREATE TABLE {table_name}
|
||||
(
|
||||
price UInt32,
|
||||
date Date,
|
||||
postcode1 LowCardinality(String),
|
||||
postcode2 LowCardinality(String),
|
||||
type Enum8('other' = 0, 'terraced' = 1, 'semi-detached' = 2, 'detached' = 3, 'flat' = 4),
|
||||
is_new UInt8,
|
||||
duration Enum8('unknown' = 0, 'freehold' = 1, 'leasehold' = 2),
|
||||
addr1 String,
|
||||
addr2 String,
|
||||
street LowCardinality(String),
|
||||
locality LowCardinality(String),
|
||||
town LowCardinality(String),
|
||||
district LowCardinality(String),
|
||||
county LowCardinality(String)
|
||||
)
|
||||
ENGINE = {engine}
|
||||
ORDER BY (postcode1, postcode2, addr1, addr2)
|
||||
""".format(
|
||||
table_name=table_name, engine=engine
|
||||
)
|
||||
)
|
||||
|
||||
|
||||
def test_both_mergtree(start_cluster):
|
||||
create_source_table(replica1, "source", False)
|
||||
create_destination_table(replica1, "destination", False)
|
||||
|
||||
replica1.query(f"ALTER TABLE destination ATTACH PARTITION tuple() FROM source")
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica1,
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM destination GROUP BY year ORDER BY year ASC",
|
||||
replica1.query(
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM source GROUP BY year ORDER BY year ASC"
|
||||
),
|
||||
)
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica1, f"SELECT town from destination LIMIT 1", "SCARBOROUGH"
|
||||
)
|
||||
|
||||
cleanup([replica1])
|
||||
|
||||
|
||||
def test_all_replicated(start_cluster):
|
||||
create_source_table(replica1, "source", True)
|
||||
create_destination_table(replica1, "destination", True)
|
||||
create_destination_table(replica2, "destination", True)
|
||||
|
||||
replica1.query("SYSTEM SYNC REPLICA destination")
|
||||
replica1.query(f"ALTER TABLE destination ATTACH PARTITION tuple() FROM source")
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica1,
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM destination GROUP BY year ORDER BY year ASC",
|
||||
replica1.query(
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM source GROUP BY year ORDER BY year ASC"
|
||||
),
|
||||
)
|
||||
assert_eq_with_retry(
|
||||
replica1,
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM source GROUP BY year ORDER BY year ASC",
|
||||
replica2.query(
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM destination GROUP BY year ORDER BY year ASC"
|
||||
),
|
||||
)
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica1, f"SELECT town from destination LIMIT 1", "SCARBOROUGH"
|
||||
)
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica2, f"SELECT town from destination LIMIT 1", "SCARBOROUGH"
|
||||
)
|
||||
|
||||
cleanup([replica1, replica2])
|
||||
|
||||
|
||||
def test_only_destination_replicated(start_cluster):
|
||||
create_source_table(replica1, "source", False)
|
||||
create_destination_table(replica1, "destination", True)
|
||||
create_destination_table(replica2, "destination", True)
|
||||
|
||||
replica1.query("SYSTEM SYNC REPLICA destination")
|
||||
replica1.query(f"ALTER TABLE destination ATTACH PARTITION tuple() FROM source")
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica1,
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM destination GROUP BY year ORDER BY year ASC",
|
||||
replica1.query(
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM source GROUP BY year ORDER BY year ASC"
|
||||
),
|
||||
)
|
||||
assert_eq_with_retry(
|
||||
replica1,
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM source GROUP BY year ORDER BY year ASC",
|
||||
replica2.query(
|
||||
f"SELECT toYear(date) AS year,round(avg(price)) AS price,bar(price, 0, 1000000, 80) FROM destination GROUP BY year ORDER BY year ASC"
|
||||
),
|
||||
)
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica1, f"SELECT town from destination LIMIT 1", "SCARBOROUGH"
|
||||
)
|
||||
|
||||
assert_eq_with_retry(
|
||||
replica2, f"SELECT town from destination LIMIT 1", "SCARBOROUGH"
|
||||
)
|
||||
|
||||
cleanup([replica1, replica2])
|
@ -41,6 +41,38 @@ def generate_cluster_def(port):
|
||||
<account_key>Eby8vdM02xNOcqFlqUwJPLlmEtlCDXJ1OUzFT50uSRZ6IFsuFq2UVErCz4I6tq/K1SZFPTOtr/KBHBeksoGMGw==</account_key>
|
||||
</azure_conf2>
|
||||
</named_collections>
|
||||
<storage_configuration>
|
||||
<disks>
|
||||
<blob_storage_disk>
|
||||
<type>azure_blob_storage</type>
|
||||
<storage_account_url>http://azurite1:{port}/devstoreaccount1</storage_account_url>
|
||||
<container_name>cont</container_name>
|
||||
<skip_access_check>false</skip_access_check>
|
||||
<account_name>devstoreaccount1</account_name>
|
||||
<account_key>Eby8vdM02xNOcqFlqUwJPLlmEtlCDXJ1OUzFT50uSRZ6IFsuFq2UVErCz4I6tq/K1SZFPTOtr/KBHBeksoGMGw==</account_key>
|
||||
<max_single_part_upload_size>100000</max_single_part_upload_size>
|
||||
<min_upload_part_size>100000</min_upload_part_size>
|
||||
<max_single_download_retries>10</max_single_download_retries>
|
||||
<max_single_read_retries>10</max_single_read_retries>
|
||||
</blob_storage_disk>
|
||||
<hdd>
|
||||
<type>local</type>
|
||||
<path>/</path>
|
||||
</hdd>
|
||||
</disks>
|
||||
<policies>
|
||||
<blob_storage_policy>
|
||||
<volumes>
|
||||
<main>
|
||||
<disk>blob_storage_disk</disk>
|
||||
</main>
|
||||
<external>
|
||||
<disk>hdd</disk>
|
||||
</external>
|
||||
</volumes>
|
||||
</blob_storage_policy>
|
||||
</policies>
|
||||
</storage_configuration>
|
||||
</clickhouse>
|
||||
"""
|
||||
)
|
||||
@ -169,12 +201,12 @@ def test_backup_restore(cluster):
|
||||
print(get_azure_file_content("test_simple_write_c.csv", port))
|
||||
assert get_azure_file_content("test_simple_write_c.csv", port) == '1,"a"\n'
|
||||
|
||||
backup_destination = f"AzureBlobStorage('{cluster.env_variables['AZURITE_CONNECTION_STRING']}', 'cont', 'test_simple_write_c_backup.csv')"
|
||||
backup_destination = f"AzureBlobStorage('{cluster.env_variables['AZURITE_CONNECTION_STRING']}', 'cont', 'test_simple_write_c_backup')"
|
||||
azure_query(
|
||||
node,
|
||||
f"BACKUP TABLE test_simple_write_connection_string TO {backup_destination}",
|
||||
)
|
||||
print(get_azure_file_content("test_simple_write_c_backup.csv.backup", port))
|
||||
print(get_azure_file_content("test_simple_write_c_backup/.backup", port))
|
||||
azure_query(
|
||||
node,
|
||||
f"RESTORE TABLE test_simple_write_connection_string AS test_simple_write_connection_string_restored FROM {backup_destination};",
|
||||
@ -195,7 +227,7 @@ def test_backup_restore_diff_container(cluster):
|
||||
azure_query(
|
||||
node, f"INSERT INTO test_simple_write_connection_string_cont1 VALUES (1, 'a')"
|
||||
)
|
||||
backup_destination = f"AzureBlobStorage('{cluster.env_variables['AZURITE_CONNECTION_STRING']}', 'cont1', 'test_simple_write_c_backup_cont1.csv')"
|
||||
backup_destination = f"AzureBlobStorage('{cluster.env_variables['AZURITE_CONNECTION_STRING']}', 'cont1', 'test_simple_write_c_backup_cont1')"
|
||||
azure_query(
|
||||
node,
|
||||
f"BACKUP TABLE test_simple_write_connection_string_cont1 TO {backup_destination}",
|
||||
@ -223,14 +255,12 @@ def test_backup_restore_with_named_collection_azure_conf1(cluster):
|
||||
print(get_azure_file_content("test_simple_write.csv", port))
|
||||
assert get_azure_file_content("test_simple_write.csv", port) == '1,"a"\n'
|
||||
|
||||
backup_destination = (
|
||||
f"AzureBlobStorage(azure_conf1, 'test_simple_write_nc_backup.csv')"
|
||||
)
|
||||
backup_destination = f"AzureBlobStorage(azure_conf1, 'test_simple_write_nc_backup')"
|
||||
azure_query(
|
||||
node,
|
||||
f"BACKUP TABLE test_write_connection_string TO {backup_destination}",
|
||||
)
|
||||
print(get_azure_file_content("test_simple_write_nc_backup.csv.backup", port))
|
||||
print(get_azure_file_content("test_simple_write_nc_backup/.backup", port))
|
||||
azure_query(
|
||||
node,
|
||||
f"RESTORE TABLE test_write_connection_string AS test_write_connection_string_restored FROM {backup_destination};",
|
||||
@ -253,13 +283,13 @@ def test_backup_restore_with_named_collection_azure_conf2(cluster):
|
||||
assert get_azure_file_content("test_simple_write_2.csv", port) == '1,"a"\n'
|
||||
|
||||
backup_destination = (
|
||||
f"AzureBlobStorage(azure_conf2, 'test_simple_write_nc_backup_2.csv')"
|
||||
f"AzureBlobStorage(azure_conf2, 'test_simple_write_nc_backup_2')"
|
||||
)
|
||||
azure_query(
|
||||
node,
|
||||
f"BACKUP TABLE test_write_connection_string_2 TO {backup_destination}",
|
||||
)
|
||||
print(get_azure_file_content("test_simple_write_nc_backup_2.csv.backup", port))
|
||||
print(get_azure_file_content("test_simple_write_nc_backup_2/.backup", port))
|
||||
azure_query(
|
||||
node,
|
||||
f"RESTORE TABLE test_write_connection_string_2 AS test_write_connection_string_restored_2 FROM {backup_destination};",
|
||||
@ -268,3 +298,26 @@ def test_backup_restore_with_named_collection_azure_conf2(cluster):
|
||||
azure_query(node, f"SELECT * from test_write_connection_string_restored_2")
|
||||
== "1\ta\n"
|
||||
)
|
||||
|
||||
|
||||
def test_backup_restore_on_merge_tree(cluster):
|
||||
node = cluster.instances["node"]
|
||||
port = cluster.env_variables["AZURITE_PORT"]
|
||||
azure_query(
|
||||
node,
|
||||
f"CREATE TABLE test_simple_merge_tree(key UInt64, data String) Engine = MergeTree() ORDER BY tuple() SETTINGS storage_policy='blob_storage_policy'",
|
||||
)
|
||||
azure_query(node, f"INSERT INTO test_simple_merge_tree VALUES (1, 'a')")
|
||||
|
||||
backup_destination = f"AzureBlobStorage('{cluster.env_variables['AZURITE_CONNECTION_STRING']}', 'cont', 'test_simple_merge_tree_backup')"
|
||||
azure_query(
|
||||
node,
|
||||
f"BACKUP TABLE test_simple_merge_tree TO {backup_destination}",
|
||||
)
|
||||
azure_query(
|
||||
node,
|
||||
f"RESTORE TABLE test_simple_merge_tree AS test_simple_merge_tree_restored FROM {backup_destination};",
|
||||
)
|
||||
assert (
|
||||
azure_query(node, f"SELECT * from test_simple_merge_tree_restored") == "1\ta\n"
|
||||
)
|
||||
|
@ -5,7 +5,6 @@ import string
|
||||
import threading
|
||||
import time
|
||||
from multiprocessing.dummy import Pool
|
||||
from helpers.test_tools import assert_eq_with_retry
|
||||
|
||||
import pytest
|
||||
from helpers.client import QueryRuntimeException
|
||||
@ -1746,9 +1745,9 @@ def test_move_while_merge(start_cluster):
|
||||
node1.query(f"DROP TABLE IF EXISTS {name} SYNC")
|
||||
|
||||
|
||||
def test_move_across_policies_work_for_attach_not_work_for_move(start_cluster):
|
||||
def test_move_across_policies_does_not_work(start_cluster):
|
||||
try:
|
||||
name = "test_move_across_policies_work_for_attach_not_work_for_move"
|
||||
name = "test_move_across_policies_does_not_work"
|
||||
|
||||
node1.query(
|
||||
"""
|
||||
@ -1784,18 +1783,25 @@ def test_move_across_policies_work_for_attach_not_work_for_move(start_cluster):
|
||||
except QueryRuntimeException:
|
||||
"""All parts of partition 'all' are already on disk 'jbod2'."""
|
||||
|
||||
node1.query(
|
||||
"""ALTER TABLE {name}2 ATTACH PARTITION tuple() FROM {name}""".format(
|
||||
name=name
|
||||
)
|
||||
)
|
||||
assert_eq_with_retry(
|
||||
node1,
|
||||
"""SELECT * FROM {name}2""".format(name=name),
|
||||
with pytest.raises(
|
||||
QueryRuntimeException,
|
||||
match=".*because disk does not belong to storage policy.*",
|
||||
):
|
||||
node1.query(
|
||||
"""SELECT * FROM {name}""".format(name=name),
|
||||
),
|
||||
)
|
||||
"""ALTER TABLE {name}2 ATTACH PARTITION tuple() FROM {name}""".format(
|
||||
name=name
|
||||
)
|
||||
)
|
||||
|
||||
with pytest.raises(
|
||||
QueryRuntimeException,
|
||||
match=".*because disk does not belong to storage policy.*",
|
||||
):
|
||||
node1.query(
|
||||
"""ALTER TABLE {name}2 REPLACE PARTITION tuple() FROM {name}""".format(
|
||||
name=name
|
||||
)
|
||||
)
|
||||
|
||||
with pytest.raises(
|
||||
QueryRuntimeException,
|
||||
@ -1807,6 +1813,10 @@ def test_move_across_policies_work_for_attach_not_work_for_move(start_cluster):
|
||||
)
|
||||
)
|
||||
|
||||
assert node1.query(
|
||||
"""SELECT * FROM {name}""".format(name=name)
|
||||
).splitlines() == ["1"]
|
||||
|
||||
finally:
|
||||
node1.query(f"DROP TABLE IF EXISTS {name} SYNC")
|
||||
node1.query(f"DROP TABLE IF EXISTS {name}2 SYNC")
|
||||
|
@ -20,7 +20,7 @@ expect_after {
|
||||
-i $any_spawn_id timeout { exit 1 }
|
||||
}
|
||||
|
||||
spawn bash -c "source $basedir/../shell_config.sh ; \$CLICKHOUSE_CLIENT_BINARY \$CLICKHOUSE_CLIENT_OPT --history_file=$history_file"
|
||||
spawn bash -c "source $basedir/../shell_config.sh ; \$CLICKHOUSE_CLIENT_BINARY \$CLICKHOUSE_CLIENT_OPT --history_file=$history_file --highlight=0"
|
||||
expect ":) "
|
||||
|
||||
# Make a query
|
||||
|
@ -24,30 +24,21 @@ expect_after {
|
||||
-i $any_spawn_id timeout { exit 1 }
|
||||
}
|
||||
|
||||
spawn bash -c "source $basedir/../shell_config.sh ; \$CLICKHOUSE_CLIENT_BINARY \$CLICKHOUSE_CLIENT_OPT --disable_suggestion -mn --history_file=$history_file"
|
||||
spawn bash -c "source $basedir/../shell_config.sh ; \$CLICKHOUSE_CLIENT_BINARY \$CLICKHOUSE_CLIENT_OPT --disable_suggestion -mn --history_file=$history_file --highlight 0"
|
||||
expect "\n:) "
|
||||
|
||||
send -- "DROP TABLE IF EXISTS t01565;\n"
|
||||
# NOTE: this is important for -mn mode, you should send "\r" only after reading echoed command
|
||||
expect "\r\n"
|
||||
send -- "\r"
|
||||
send -- "DROP TABLE IF EXISTS t01565;\r"
|
||||
expect "\nOk."
|
||||
expect "\n:)"
|
||||
|
||||
send -- "CREATE TABLE t01565 (c0 String, c1 Int32) ENGINE = Memory() ;\n"
|
||||
expect "\r\n"
|
||||
send -- "\r"
|
||||
send -- "CREATE TABLE t01565 (c0 String, c1 Int32) ENGINE = Memory() ;\r"
|
||||
expect "\nOk."
|
||||
expect "\n:) "
|
||||
|
||||
send -- "INSERT INTO t01565(c0, c1) VALUES (\"1\",1) ;\n"
|
||||
expect "\r\n"
|
||||
send -- "\r"
|
||||
send -- "INSERT INTO t01565(c0, c1) VALUES (\"1\",1) ;\r"
|
||||
expect "\n:) "
|
||||
|
||||
send -- "INSERT INTO t01565(c0, c1) VALUES ('1', 1) ;\n"
|
||||
expect "\r\n"
|
||||
send -- "\r"
|
||||
send -- "INSERT INTO t01565(c0, c1) VALUES ('1', 1) ;\r"
|
||||
expect "\nOk."
|
||||
expect "\n:) "
|
||||
|
||||
|
@ -0,0 +1,149 @@
|
||||
import pty
|
||||
import os
|
||||
import shlex
|
||||
import time
|
||||
import multiprocessing
|
||||
|
||||
COMPLETION_TIMEOUT_SECONDS = 30
|
||||
DEBUG_LOG = os.path.join(
|
||||
os.environ["CLICKHOUSE_TMP"],
|
||||
os.path.basename(os.path.abspath(__file__)).strip(".python") + ".debuglog",
|
||||
)
|
||||
|
||||
|
||||
def run_with_timeout(func, args, timeout):
|
||||
process = multiprocessing.Process(target=func, args=args)
|
||||
process.start()
|
||||
process.join(timeout)
|
||||
|
||||
if process.is_alive():
|
||||
process.terminate()
|
||||
print("Timeout")
|
||||
|
||||
|
||||
def test_completion(program, argv, comp_word):
|
||||
comp_begin = comp_word[:-3]
|
||||
|
||||
shell_pid, master = pty.fork()
|
||||
if shell_pid == 0:
|
||||
os.execv(program, argv)
|
||||
else:
|
||||
try:
|
||||
debug_log_fd = open(DEBUG_LOG, "a")
|
||||
|
||||
output_b = os.read(master, 4096)
|
||||
output = output_b.decode()
|
||||
debug_log_fd.write(repr(output_b) + "\n")
|
||||
debug_log_fd.flush()
|
||||
while not ":)" in output:
|
||||
output_b = os.read(master, 4096)
|
||||
output += output_b.decode()
|
||||
debug_log_fd.write(repr(output_b) + "\n")
|
||||
debug_log_fd.flush()
|
||||
|
||||
os.write(master, b"SET " + bytes(comp_begin.encode()))
|
||||
output_b = os.read(master, 4096)
|
||||
output = output_b.decode()
|
||||
debug_log_fd.write(repr(output_b) + "\n")
|
||||
debug_log_fd.flush()
|
||||
while not comp_begin in output:
|
||||
output_b = os.read(master, 4096)
|
||||
output += output_b.decode()
|
||||
debug_log_fd.write(repr(output_b) + "\n")
|
||||
debug_log_fd.flush()
|
||||
|
||||
time.sleep(0.01)
|
||||
os.write(master, b"\t")
|
||||
|
||||
output_b = os.read(master, 4096)
|
||||
output = output_b.decode()
|
||||
debug_log_fd.write(repr(output_b) + "\n")
|
||||
debug_log_fd.flush()
|
||||
# fail fast if there is a bell character in the output,
|
||||
# meaning no concise completion is found
|
||||
if "\x07" in output:
|
||||
print(f"{comp_word}: FAIL")
|
||||
return
|
||||
|
||||
while not comp_word in output:
|
||||
output_b = os.read(master, 4096)
|
||||
output += output_b.decode()
|
||||
debug_log_fd.write(repr(output_b) + "\n")
|
||||
debug_log_fd.flush()
|
||||
|
||||
print(f"{comp_word}: OK")
|
||||
finally:
|
||||
os.close(master)
|
||||
debug_log_fd.close()
|
||||
|
||||
|
||||
client_compwords_positive = [
|
||||
# system.functions
|
||||
"concatAssumeInjective",
|
||||
# system.table_engines
|
||||
"ReplacingMergeTree",
|
||||
# system.formats
|
||||
"JSONEachRow",
|
||||
# system.table_functions
|
||||
"clusterAllReplicas",
|
||||
# system.data_type_families
|
||||
"SimpleAggregateFunction",
|
||||
# system.settings
|
||||
"max_concurrent_queries_for_all_users",
|
||||
# system.clusters
|
||||
"test_shard_localhost",
|
||||
# system.macros
|
||||
"default_path_test",
|
||||
# system.storage_policies, egh not uniq
|
||||
"default",
|
||||
# system.aggregate_function_combinators
|
||||
"uniqCombined64ForEach",
|
||||
# FIXME: one may add separate case for suggestion_limit
|
||||
# system.databases
|
||||
"system",
|
||||
# system.tables
|
||||
"aggregate_function_combinators",
|
||||
# system.columns
|
||||
"primary_key_bytes_in_memory_allocated",
|
||||
# system.dictionaries
|
||||
# FIXME: none
|
||||
]
|
||||
|
||||
local_compwords_positive = [
|
||||
# system.functions
|
||||
"concatAssumeInjective",
|
||||
# system.table_engines
|
||||
"ReplacingMergeTree",
|
||||
# system.formats
|
||||
"JSONEachRow",
|
||||
# system.table_functions
|
||||
"clusterAllReplicas",
|
||||
# system.data_type_families
|
||||
"SimpleAggregateFunction",
|
||||
]
|
||||
|
||||
|
||||
if __name__ == "__main__":
|
||||
print("# clickhouse-client")
|
||||
clickhouse_client = os.environ["CLICKHOUSE_CLIENT"]
|
||||
args = shlex.split(clickhouse_client)
|
||||
args.append("--wait_for_suggestions_to_load")
|
||||
args.append("--highlight=0")
|
||||
[
|
||||
run_with_timeout(
|
||||
test_completion, [args[0], args, comp_word], COMPLETION_TIMEOUT_SECONDS
|
||||
)
|
||||
for comp_word in client_compwords_positive
|
||||
]
|
||||
|
||||
print("# clickhouse-local")
|
||||
clickhouse_local = os.environ["CLICKHOUSE_LOCAL"]
|
||||
args = shlex.split(clickhouse_local)
|
||||
args.append("--wait_for_suggestions_to_load")
|
||||
args.append("--highlight=0")
|
||||
[
|
||||
run_with_timeout(
|
||||
test_completion, [args[0], args, comp_word], COMPLETION_TIMEOUT_SECONDS
|
||||
)
|
||||
for comp_word in local_compwords_positive
|
||||
]
|
@ -1,142 +1,8 @@
|
||||
#!/usr/bin/env bash
|
||||
# Tags: long, no-ubsan
|
||||
# Tags: long
|
||||
|
||||
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
. "$CURDIR"/../shell_config.sh
|
||||
|
||||
SCRIPT_PATH="$CURDIR/$CLICKHOUSE_TEST_UNIQUE_NAME.generated-expect"
|
||||
|
||||
# NOTE: database = $CLICKHOUSE_DATABASE is superfluous
|
||||
|
||||
function test_completion_word()
|
||||
{
|
||||
local w=$1 && shift
|
||||
|
||||
local w_len=${#w}
|
||||
local compword_begin=${w:0:$((w_len-3))}
|
||||
local compword_end=${w:$((w_len-3))}
|
||||
|
||||
# NOTE:
|
||||
# - here and below you should escape variables of the expect.
|
||||
# - you should not use "expect <<..." since in this case timeout/eof will
|
||||
# not work (I guess due to attached stdin)
|
||||
|
||||
# TODO: get build sanitizer and debug/release info to dynamically change test
|
||||
# like here timeout 120 seconds is too big for release build
|
||||
# but ok for sanitizer builds
|
||||
cat > "$SCRIPT_PATH" << EOF
|
||||
# NOTE: log will be appended
|
||||
exp_internal -f $CLICKHOUSE_TMP/$(basename "${BASH_SOURCE[0]}").debuglog 0
|
||||
|
||||
# NOTE: when expect have EOF on stdin it also closes stdout, so let's reopen it
|
||||
# again for logging
|
||||
set stdout_channel [open "/dev/stdout" w]
|
||||
|
||||
log_user 0
|
||||
set timeout 120
|
||||
match_max 100000
|
||||
expect_after {
|
||||
# Do not ignore eof from expect
|
||||
-i \$any_spawn_id eof { exp_continue }
|
||||
# A default timeout action is to do nothing, change it to fail
|
||||
-i \$any_spawn_id timeout { exit 1 }
|
||||
}
|
||||
|
||||
spawn bash -c "$*"
|
||||
expect ":) "
|
||||
|
||||
# Make a query
|
||||
send -- "SET $compword_begin"
|
||||
expect "SET $compword_begin"
|
||||
|
||||
# Wait for suggestions to load, they are loaded in background
|
||||
set is_done 0
|
||||
set timeout 1
|
||||
while {\$is_done == 0} {
|
||||
send -- "\\t"
|
||||
expect {
|
||||
"$compword_begin$compword_end" {
|
||||
puts \$stdout_channel "$compword_begin$compword_end: OK"
|
||||
set is_done 1
|
||||
}
|
||||
default {
|
||||
sleep 1
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
close \$stdout_channel
|
||||
|
||||
send -- "\\3\\4"
|
||||
expect eof
|
||||
EOF
|
||||
|
||||
# NOTE: run expect under timeout since there is while loop that is not
|
||||
# limited with timeout.
|
||||
#
|
||||
# NOTE: cat is required to serialize stdout for expect (without this pipe
|
||||
# it will reopen the file again, and the output will be mixed).
|
||||
timeout 2m expect -f "$SCRIPT_PATH" | cat
|
||||
}
|
||||
|
||||
# last 3 bytes will be completed,
|
||||
# so take this in mind when you will update the list.
|
||||
client_compwords_positive=(
|
||||
# system.functions
|
||||
concatAssumeInjective
|
||||
# system.table_engines
|
||||
ReplacingMergeTree
|
||||
# system.formats
|
||||
JSONEachRow
|
||||
# system.table_functions
|
||||
clusterAllReplicas
|
||||
# system.data_type_families
|
||||
SimpleAggregateFunction
|
||||
# system.settings
|
||||
max_concurrent_queries_for_all_users
|
||||
# system.clusters
|
||||
test_shard_localhost
|
||||
# system.macros
|
||||
default_path_test
|
||||
# system.storage_policies, egh not uniq
|
||||
default
|
||||
# system.aggregate_function_combinators
|
||||
uniqCombined64ForEach
|
||||
|
||||
# FIXME: one may add separate case for suggestion_limit
|
||||
# system.databases
|
||||
system
|
||||
# system.tables
|
||||
aggregate_function_combinators
|
||||
# system.columns
|
||||
primary_key_bytes_in_memory_allocated
|
||||
# system.dictionaries
|
||||
# FIXME: none
|
||||
)
|
||||
|
||||
local_compwords_positive=(
|
||||
# system.functions
|
||||
concatAssumeInjective
|
||||
# system.table_engines
|
||||
ReplacingMergeTree
|
||||
# system.formats
|
||||
JSONEachRow
|
||||
# system.table_functions
|
||||
clusterAllReplicas
|
||||
# system.data_type_families
|
||||
SimpleAggregateFunction
|
||||
)
|
||||
|
||||
echo "# clickhouse-client"
|
||||
for w in "${client_compwords_positive[@]}"; do
|
||||
test_completion_word "$w" "$CLICKHOUSE_CLIENT"
|
||||
done
|
||||
echo "# clickhouse-local"
|
||||
for w in "${local_compwords_positive[@]}"; do
|
||||
test_completion_word "$w" "$CLICKHOUSE_LOCAL"
|
||||
done
|
||||
|
||||
rm -f "${SCRIPT_PATH:?}"
|
||||
|
||||
exit 0
|
||||
python3 "$CURDIR"/01676_clickhouse_client_autocomplete.python
|
||||
|
@ -43,16 +43,16 @@ Alter ALTER TABLE sqllt.table UPDATE i = i + 1 WHERE 1;
|
||||
Alter ALTER TABLE sqllt.table DELETE WHERE i > 65535;
|
||||
Select -- not done, seems to hard, so I\'ve skipped queries of ALTER-X, where X is:\n-- PARTITION\n-- ORDER BY\n-- SAMPLE BY\n-- INDEX\n-- CONSTRAINT\n-- TTL\n-- USER\n-- QUOTA\n-- ROLE\n-- ROW POLICY\n-- SETTINGS PROFILE\n\nSELECT \'SYSTEM queries\';
|
||||
System SYSTEM FLUSH LOGS;
|
||||
System SYSTEM STOP MERGES sqllt.table
|
||||
System SYSTEM START MERGES sqllt.table
|
||||
System SYSTEM STOP TTL MERGES sqllt.table
|
||||
System SYSTEM START TTL MERGES sqllt.table
|
||||
System SYSTEM STOP MOVES sqllt.table
|
||||
System SYSTEM START MOVES sqllt.table
|
||||
System SYSTEM STOP FETCHES sqllt.table
|
||||
System SYSTEM START FETCHES sqllt.table
|
||||
System SYSTEM STOP REPLICATED SENDS sqllt.table
|
||||
System SYSTEM START REPLICATED SENDS sqllt.table
|
||||
System SYSTEM STOP MERGES sqllt.table;
|
||||
System SYSTEM START MERGES sqllt.table;
|
||||
System SYSTEM STOP TTL MERGES sqllt.table;
|
||||
System SYSTEM START TTL MERGES sqllt.table;
|
||||
System SYSTEM STOP MOVES sqllt.table;
|
||||
System SYSTEM START MOVES sqllt.table;
|
||||
System SYSTEM STOP FETCHES sqllt.table;
|
||||
System SYSTEM START FETCHES sqllt.table;
|
||||
System SYSTEM STOP REPLICATED SENDS sqllt.table;
|
||||
System SYSTEM START REPLICATED SENDS sqllt.table;
|
||||
Select -- SYSTEM RELOAD DICTIONARY sqllt.dictionary; -- temporary out of order: Code: 210, Connection refused (localhost:9001) (version 21.3.1.1)\n-- DROP REPLICA\n-- haha, no\n-- SYSTEM KILL;\n-- SYSTEM SHUTDOWN;\n\n-- Since we don\'t really care about the actual output, suppress it with `FORMAT Null`.\nSELECT \'SHOW queries\';
|
||||
Show SHOW CREATE TABLE sqllt.table FORMAT Null;
|
||||
Show SHOW CREATE DICTIONARY sqllt.dictionary FORMAT Null;
|
||||
|
@ -21,7 +21,7 @@ expect_after {
|
||||
-i $any_spawn_id timeout { exit 1 }
|
||||
}
|
||||
|
||||
spawn bash -c "source $basedir/../shell_config.sh ; \$CLICKHOUSE_CLIENT_BINARY \$CLICKHOUSE_CLIENT_OPT --history_file=$history_file"
|
||||
spawn bash -c "source $basedir/../shell_config.sh ; \$CLICKHOUSE_CLIENT_BINARY \$CLICKHOUSE_CLIENT_OPT --history_file=$history_file --highlight=0"
|
||||
expect ":) "
|
||||
|
||||
# Make a query
|
||||
|
32
tests/queries/0_stateless/03032_redundant_equals.reference
Normal file
32
tests/queries/0_stateless/03032_redundant_equals.reference
Normal file
@ -0,0 +1,32 @@
|
||||
100
|
||||
100
|
||||
100
|
||||
100
|
||||
100
|
||||
100
|
||||
0
|
||||
0
|
||||
0
|
||||
1
|
||||
100
|
||||
101
|
||||
100
|
||||
101
|
||||
100
|
||||
101
|
||||
100
|
||||
101
|
||||
101
|
||||
101
|
||||
100
|
||||
101
|
||||
100
|
||||
101
|
||||
100
|
||||
101
|
||||
1
|
||||
1
|
||||
1
|
||||
1
|
||||
1
|
||||
1
|
92
tests/queries/0_stateless/03032_redundant_equals.sql
Normal file
92
tests/queries/0_stateless/03032_redundant_equals.sql
Normal file
@ -0,0 +1,92 @@
|
||||
DROP TABLE IF EXISTS test_table;
|
||||
|
||||
CREATE TABLE test_table
|
||||
(
|
||||
k UInt64,
|
||||
)
|
||||
ENGINE = MergeTree
|
||||
ORDER BY k;
|
||||
|
||||
INSERT INTO test_table SELECT number FROM numbers(10000000);
|
||||
|
||||
SET allow_experimental_analyzer = 1;
|
||||
|
||||
SELECT * FROM test_table WHERE k in (100) = 1;
|
||||
SELECT * FROM test_table WHERE k = (100) = 1;
|
||||
SELECT * FROM test_table WHERE k not in (100) = 0;
|
||||
SELECT * FROM test_table WHERE k != (100) = 0;
|
||||
SELECT * FROM test_table WHERE 1 = (k = 100);
|
||||
SELECT * FROM test_table WHERE 0 = (k not in (100));
|
||||
SELECT * FROM test_table WHERE k < 1 = 1;
|
||||
SELECT * FROM test_table WHERE k >= 1 = 0;
|
||||
SELECT * FROM test_table WHERE k > 1 = 0;
|
||||
SELECT * FROM test_table WHERE ((k not in (101) = 0) OR (k in (100) = 1)) = 1;
|
||||
SELECT * FROM test_table WHERE (NOT ((k not in (100) = 0) OR (k in (100) = 1))) = 0;
|
||||
SELECT * FROM test_table WHERE (NOT ((k in (101) = 0) OR (k in (100) = 1))) = 1;
|
||||
SELECT * FROM test_table WHERE ((k not in (101) = 0) OR (k in (100) = 1)) = 1;
|
||||
SELECT * FROM test_table WHERE ((k not in (99) = 1) AND (k in (100) = 1)) = 1;
|
||||
-- we skip optimizing queries with toNullable(0 or 1) but lets make sure they still work
|
||||
SELECT * FROM test_table WHERE (k = 101) = toLowCardinality(toNullable(1));
|
||||
SELECT * FROM test_table WHERE (k = 101) = toNullable(1);
|
||||
SELECT * FROM test_table WHERE (k = 101) = toLowCardinality(1);
|
||||
SELECT * FROM test_table WHERE ((k not in (101) = toNullable(0)) OR (k in (100) = toNullable(1))) = toNullable(1);
|
||||
SELECT * FROM test_table WHERE (((k NOT IN toLowCardinality(toNullable(101))) = toLowCardinality(toNullable(0))) OR ((k IN (toLowCardinality(100))) = toNullable(1)));
|
||||
SELECT * FROM test_table WHERE (((k IN toLowCardinality(toNullable(101))) = toLowCardinality(toNullable(0))) AND ((k NOT IN (toLowCardinality(100))) = toNullable(1))) = toNullable(toLowCardinality(0));
|
||||
|
||||
SELECT count()
|
||||
FROM
|
||||
(
|
||||
EXPLAIN PLAN indexes=1
|
||||
SELECT * FROM test_table WHERE k in (100) = 1
|
||||
)
|
||||
WHERE
|
||||
explain LIKE '%Granules: 1/%';
|
||||
|
||||
SELECT count()
|
||||
FROM
|
||||
(
|
||||
EXPLAIN PLAN indexes=1
|
||||
SELECT * FROM test_table WHERE k >= 1 = 0
|
||||
)
|
||||
WHERE
|
||||
explain LIKE '%Granules: 1/%';
|
||||
|
||||
SELECT count()
|
||||
FROM
|
||||
(
|
||||
EXPLAIN PLAN indexes=1
|
||||
SELECT * FROM test_table WHERE k not in (100) = 0
|
||||
)
|
||||
WHERE
|
||||
explain LIKE '%Granules: 1/%';
|
||||
|
||||
SELECT count()
|
||||
FROM
|
||||
(
|
||||
EXPLAIN PLAN indexes=1
|
||||
SELECT * FROM test_table WHERE k > 1 = 0
|
||||
)
|
||||
WHERE
|
||||
explain LIKE '%Granules: 1/%';
|
||||
|
||||
SELECT count()
|
||||
FROM
|
||||
(
|
||||
EXPLAIN PLAN indexes=1
|
||||
SELECT * FROM test_table WHERE (NOT ((k not in (100) = 0) OR (k in (100) = 1))) = 0
|
||||
)
|
||||
WHERE
|
||||
explain LIKE '%Granules: 1/%';
|
||||
|
||||
|
||||
SELECT count()
|
||||
FROM
|
||||
(
|
||||
EXPLAIN PLAN indexes=1
|
||||
SELECT * FROM test_table WHERE (NOT ((k in (101) = 0) OR (k in (100) = 1))) = 1
|
||||
)
|
||||
WHERE
|
||||
explain LIKE '%Granules: 1/%';
|
||||
|
||||
|
||||
DROP TABLE test_table;
|
@ -0,0 +1 @@
|
||||
1
|
@ -0,0 +1,3 @@
|
||||
DROP TABLE IF EXISTS override_test;
|
||||
CREATE TABLE override_test (_part UInt32) ENGINE = MergeTree ORDER BY tuple() AS SELECT 1;
|
||||
SELECT _part FROM override_test;
|
@ -0,0 +1 @@
|
||||
1 0 10 9
|
21
tests/queries/0_stateless/03093_analyzer_column_alias.sql
Normal file
21
tests/queries/0_stateless/03093_analyzer_column_alias.sql
Normal file
@ -0,0 +1,21 @@
|
||||
-- https://github.com/ClickHouse/ClickHouse/issues/26674
|
||||
SET allow_experimental_analyzer = true;
|
||||
|
||||
SELECT
|
||||
Carrier,
|
||||
sum(toFloat64(C3)) AS C1,
|
||||
sum(toFloat64(C1)) AS C2,
|
||||
sum(toFloat64(C2)) AS C3
|
||||
FROM
|
||||
(
|
||||
SELECT
|
||||
1 AS Carrier,
|
||||
count(CAST(1, 'Nullable(Int32)')) AS C1,
|
||||
max(number) AS C2,
|
||||
min(number) AS C3
|
||||
FROM numbers(10)
|
||||
GROUP BY Carrier
|
||||
) AS ITBL
|
||||
GROUP BY Carrier
|
||||
LIMIT 1000001
|
||||
SETTINGS prefer_column_name_to_alias=1;
|
@ -0,0 +1 @@
|
||||
1
|
@ -0,0 +1,2 @@
|
||||
CREATE TABLE override_test__fuzz_45 (`_part` Float32) ENGINE = MergeTree ORDER BY tuple() AS SELECT 1;
|
||||
SELECT _part FROM override_test__fuzz_45 GROUP BY materialize(6), 1;
|
@ -0,0 +1 @@
|
||||
1
|
9
tests/queries/0_stateless/03094_recursive_type_proto.sh
Executable file
9
tests/queries/0_stateless/03094_recursive_type_proto.sh
Executable file
@ -0,0 +1,9 @@
|
||||
#!/usr/bin/env bash
|
||||
# Tags: no-fasttest
|
||||
|
||||
CUR_DIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
. "$CUR_DIR"/../shell_config.sh
|
||||
|
||||
SCHEMADIR=$CURDIR/format_schemas
|
||||
$CLICKHOUSE_LOCAL -q "DESCRIBE TABLE file('nonexist', 'Protobuf') SETTINGS format_schema='$SCHEMADIR/03094_recursive_type.proto:Struct'" |& grep -c CANNOT_PARSE_PROTOBUF_SCHEMA
|
@ -0,0 +1,4 @@
|
||||
1
|
||||
1
|
||||
(2,2)
|
||||
2
|
@ -0,0 +1,7 @@
|
||||
SELECT transform(1, [1, 1, 1], [1, 4, 5]);
|
||||
SELECT transform('1', ['1', '1', '1'], ['1', '4', '5']);
|
||||
SELECT transform((0, 0), [(0, 0), (0, 0), (0, 0)], [(2, 2), (5, 5), (10, 10)]);
|
||||
|
||||
-- https://github.com/ClickHouse/ClickHouse/issues/62183
|
||||
-- Case is turned into caseWithExpression, which then it's turned into transform
|
||||
select case 1 when 1 then 2 when 1 then 4 end;
|
@ -0,0 +1,17 @@
|
||||
syntax = "proto3";
|
||||
|
||||
message Struct {
|
||||
map<string, Value> fields = 1;
|
||||
}
|
||||
|
||||
message Value {
|
||||
// The kind of value.
|
||||
oneof kind {
|
||||
string string_value = 1;
|
||||
ListValue list_value = 2;
|
||||
}
|
||||
}
|
||||
|
||||
message ListValue {
|
||||
repeated Value values = 1;
|
||||
}
|
Loading…
Reference in New Issue
Block a user