Merge branch 'ClickHouse:master' into time_buckets_impl

This commit is contained in:
Yarik Briukhovetskyi 2024-06-07 13:24:46 +02:00 committed by GitHub
commit f6d525f6df
No known key found for this signature in database
GPG Key ID: B5690EEEBB952194
99 changed files with 1564 additions and 338 deletions

View File

@ -19,3 +19,7 @@ charset = utf-8
indent_style = space
indent_size = 4
trim_trailing_whitespace = true
# Some SQL results have trailing whitespace which is removed by IDEs
[tests/queries/**.reference]
trim_trailing_whitespace = false

2
contrib/orc vendored

@ -1 +1 @@
Subproject commit e24f2c2a3ca0769c96704ab20ad6f512a83ea2ad
Subproject commit 947cebaf9432d708253ac08dc3012daa6b4ede6f

View File

@ -30,6 +30,7 @@ RUN pip3 install \
mypy==1.8.0 \
pylint==3.1.0 \
python-magic==0.4.24 \
flake8==4.0.1 \
requests \
thefuzz \
types-requests \

View File

@ -9,6 +9,8 @@ echo "Check style" | ts
./check-style -n |& tee /test_output/style_output.txt
echo "Check python formatting with black" | ts
./check-black -n |& tee /test_output/black_output.txt
echo "Check python with flake8" | ts
./check-flake8 |& tee /test_output/flake8_output.txt
echo "Check python type hinting with mypy" | ts
./check-mypy -n |& tee /test_output/mypy_output.txt
echo "Check typos" | ts

View File

@ -91,6 +91,9 @@ cd ./utils/check-style
# Check python type hinting with mypy
./check-mypy
# Check python with flake8
./check-flake8
# Check code with codespell
./check-typos

View File

@ -3172,7 +3172,7 @@ Default value: `0`.
## lightweight_deletes_sync {#lightweight_deletes_sync}
The same as 'mutation_sync', but controls only execution of lightweight deletes.
The same as 'mutation_sync', but controls only execution of lightweight deletes.
Possible values:
@ -4616,6 +4616,16 @@ Read more about [memory overcommit](memory-overcommit.md).
Default value: `1GiB`.
## max_untracked_memory {#max_untracked_memory}
Small allocations and deallocations are grouped in thread local variable and tracked or profiled only when amount (in absolute value) becomes larger than specified value. If the value is higher than 'memory_profiler_step' it will be effectively lowered to 'memory_profiler_step'.
Default value: `4MiB`.
## min_untracked_memory {#min_untracked_memory}
Lower bound for untracked memory limit which is applied to threads with low memory consumption. Untracked memory limit equals thread memory usage divided by 16 and clamped between `min_untracked_memory` and `max_untracked_memory` for every thread. It guarantees that total untracked memory does not exceed 10% of current memory footprint even with a lot of small threads. To disable dynamic limit for untracked memory set value `4MiB`.
Default value: `4KiB`.
## Schema Inference settings
See [schema inference](../../interfaces/schema-inference.md#schema-inference-modes) documentation for more details.

View File

@ -24,6 +24,8 @@ Alias: `lttb`.
- `x` — x coordinate. [Integer](../../../sql-reference/data-types/int-uint.md) , [Float](../../../sql-reference/data-types/float.md) , [Decimal](../../../sql-reference/data-types/decimal.md) , [Date](../../../sql-reference/data-types/date.md), [Date32](../../../sql-reference/data-types/date32.md), [DateTime](../../../sql-reference/data-types/datetime.md), [DateTime64](../../../sql-reference/data-types/datetime64.md).
- `y` — y coordinate. [Integer](../../../sql-reference/data-types/int-uint.md) , [Float](../../../sql-reference/data-types/float.md) , [Decimal](../../../sql-reference/data-types/decimal.md) , [Date](../../../sql-reference/data-types/date.md), [Date32](../../../sql-reference/data-types/date32.md), [DateTime](../../../sql-reference/data-types/datetime.md), [DateTime64](../../../sql-reference/data-types/datetime64.md).
NaNs are ignored in the provided series, meaning that any NaN values will be excluded from the analysis. This ensures that the function operates only on valid numerical data.
**Parameters**
- `n` — number of points in the resulting series. [UInt64](../../../sql-reference/data-types/int-uint.md).
@ -61,7 +63,7 @@ Result:
``` text
┌────────largestTriangleThreeBuckets(4)(x, y)───────────┐
│ [(1,10),(3,15),(5,40),(10,70)] │
│ [(1,10),(3,15),(9,55),(10,70)] │
└───────────────────────────────────────────────────────┘
```

View File

@ -2423,11 +2423,7 @@ Result:
## toUnixTimestamp64Milli
## toUnixTimestamp64Micro
## toUnixTimestamp64Nano
Converts a `DateTime64` to a `Int64` value with fixed sub-second precision. Input value is scaled up or down appropriately depending on it precision.
Converts a `DateTime64` to a `Int64` value with fixed millisecond precision. The input value is scaled up or down appropriately depending on its precision.
:::note
The output value is a timestamp in UTC, not in the timezone of `DateTime64`.
@ -2437,24 +2433,22 @@ The output value is a timestamp in UTC, not in the timezone of `DateTime64`.
```sql
toUnixTimestamp64Milli(value)
toUnixTimestamp64Micro(value)
toUnixTimestamp64Nano(value)
```
**Arguments**
- `value` — DateTime64 value with any precision.
- `value` — DateTime64 value with any precision. [DateTime64](../data-types/datetime64.md).
**Returned value**
- `value` converted to the `Int64` data type.
- `value` converted to the `Int64` data type. [Int64](../data-types/int-uint.md).
**Examples**
**Example**
Query:
```sql
WITH toDateTime64('2019-09-16 19:20:12.345678910', 6) AS dt64
WITH toDateTime64('2009-02-13 23:31:31.011', 3, 'UTC') AS dt64
SELECT toUnixTimestamp64Milli(dt64);
```
@ -2462,14 +2456,77 @@ Result:
```response
┌─toUnixTimestamp64Milli(dt64)─┐
│ 1568650812345
│ 1234567891011
└──────────────────────────────┘
```
## toUnixTimestamp64Micro
Converts a `DateTime64` to a `Int64` value with fixed microsecond precision. The input value is scaled up or down appropriately depending on its precision.
:::note
The output value is a timestamp in UTC, not in the timezone of `DateTime64`.
:::
**Syntax**
```sql
toUnixTimestamp64Micro(value)
```
**Arguments**
- `value` — DateTime64 value with any precision. [DateTime64](../data-types/datetime64.md).
**Returned value**
- `value` converted to the `Int64` data type. [Int64](../data-types/int-uint.md).
**Example**
Query:
``` sql
WITH toDateTime64('2019-09-16 19:20:12.345678910', 6) AS dt64
```sql
WITH toDateTime64('1970-01-15 06:56:07.891011', 6, 'UTC') AS dt64
SELECT toUnixTimestamp64Micro(dt64);
```
Result:
```response
┌─toUnixTimestamp64Micro(dt64)─┐
│ 1234567891011 │
└──────────────────────────────┘
```
## toUnixTimestamp64Nano
Converts a `DateTime64` to a `Int64` value with fixed nanosecond precision. The input value is scaled up or down appropriately depending on its precision.
:::note
The output value is a timestamp in UTC, not in the timezone of `DateTime64`.
:::
**Syntax**
```sql
toUnixTimestamp64Nano(value)
```
**Arguments**
- `value` — DateTime64 value with any precision. [DateTime64](../data-types/datetime64.md).
**Returned value**
- `value` converted to the `Int64` data type. [Int64](../data-types/int-uint.md).
**Example**
Query:
```sql
WITH toDateTime64('1970-01-01 00:20:34.567891011', 9, 'UTC') AS dt64
SELECT toUnixTimestamp64Nano(dt64);
```
@ -2477,34 +2534,32 @@ Result:
```response
┌─toUnixTimestamp64Nano(dt64)─┐
1568650812345678000
1234567891011
└─────────────────────────────┘
```
## fromUnixTimestamp64Milli
## fromUnixTimestamp64Micro
Converts an `Int64` to a `DateTime64` value with fixed millisecond precision and optional timezone. The input value is scaled up or down appropriately depending on its precision.
## fromUnixTimestamp64Nano
Converts an `Int64` to a `DateTime64` value with fixed sub-second precision and optional timezone. Input value is scaled up or down appropriately depending on its precision. Please note that input value is treated as UTC timestamp, not timestamp at given (or implicit) timezone.
:::note
Please note that input value is treated as a UTC timestamp, not timestamp at the given (or implicit) timezone.
:::
**Syntax**
``` sql
fromUnixTimestamp64Milli(value[, timezone])
fromUnixTimestamp64Micro(value[, timezone])
fromUnixTimestamp64Nano(value[, timezone])
```
**Arguments**
- `value``Int64` value with any precision.
- `timezone``String` (optional) timezone name of the result.
- `value` — value with any precision. [Int64](../data-types/int-uint.md).
- `timezone` — (optional) timezone name of the result. [String](../data-types/string.md).
**Returned value**
- `value` converted to the `DateTime64` data type.
- `value` converted to DateTime64 with precision `3`. [DateTime64](../data-types/datetime64.md).
**Example**
@ -2512,15 +2567,101 @@ Query:
``` sql
WITH CAST(1234567891011, 'Int64') AS i64
SELECT fromUnixTimestamp64Milli(i64, 'UTC');
SELECT
fromUnixTimestamp64Milli(i64, 'UTC') AS x,
toTypeName(x);
```
Result:
```response
┌─fromUnixTimestamp64Milli(i64, 'UTC')─┐
│ 2009-02-13 23:31:31.011 │
└──────────────────────────────────────┘
┌───────────────────────x─┬─toTypeName(x)────────┐
│ 2009-02-13 23:31:31.011 │ DateTime64(3, 'UTC') │
└─────────────────────────┴──────────────────────┘
```
## fromUnixTimestamp64Micro
Converts an `Int64` to a `DateTime64` value with fixed microsecond precision and optional timezone. The input value is scaled up or down appropriately depending on its precision.
:::note
Please note that input value is treated as a UTC timestamp, not timestamp at the given (or implicit) timezone.
:::
**Syntax**
``` sql
fromUnixTimestamp64Micro(value[, timezone])
```
**Arguments**
- `value` — value with any precision. [Int64](../data-types/int-uint.md).
- `timezone` — (optional) timezone name of the result. [String](../data-types/string.md).
**Returned value**
- `value` converted to DateTime64 with precision `6`. [DateTime64](../data-types/datetime64.md).
**Example**
Query:
``` sql
WITH CAST(1234567891011, 'Int64') AS i64
SELECT
fromUnixTimestamp64Micro(i64, 'UTC') AS x,
toTypeName(x);
```
Result:
```response
┌──────────────────────────x─┬─toTypeName(x)────────┐
│ 1970-01-15 06:56:07.891011 │ DateTime64(6, 'UTC') │
└────────────────────────────┴──────────────────────┘
```
## fromUnixTimestamp64Nano
Converts an `Int64` to a `DateTime64` value with fixed nanosecond precision and optional timezone. The input value is scaled up or down appropriately depending on its precision.
:::note
Please note that input value is treated as a UTC timestamp, not timestamp at the given (or implicit) timezone.
:::
**Syntax**
``` sql
fromUnixTimestamp64Nano(value[, timezone])
```
**Arguments**
- `value` — value with any precision. [Int64](../data-types/int-uint.md).
- `timezone` — (optional) timezone name of the result. [String](../data-types/string.md).
**Returned value**
- `value` converted to DateTime64 with precision `9`. [DateTime64](../data-types/datetime64.md).
**Example**
Query:
``` sql
WITH CAST(1234567891011, 'Int64') AS i64
SELECT
fromUnixTimestamp64Nano(i64, 'UTC') AS x,
toTypeName(x);
```
Result:
```response
┌─────────────────────────────x─┬─toTypeName(x)────────┐
│ 1970-01-01 00:20:34.567891011 │ DateTime64(9, 'UTC') │
└───────────────────────────────┴──────────────────────┘
```
## formatRow

View File

@ -1,5 +1,6 @@
#include <Analyzer/ArrayJoinNode.h>
#include <Analyzer/ColumnNode.h>
#include <Analyzer/FunctionNode.h>
#include <Analyzer/Utils.h>
#include <IO/Operators.h>
#include <IO/WriteBuffer.h>
@ -64,7 +65,12 @@ ASTPtr ArrayJoinNode::toASTImpl(const ConvertToASTOptions & options) const
auto * column_node = array_join_expression->as<ColumnNode>();
if (column_node && column_node->getExpression())
array_join_expression_ast = column_node->getExpression()->toAST(options);
{
if (const auto * function_node = column_node->getExpression()->as<FunctionNode>(); function_node && function_node->getFunctionName() == "nested")
array_join_expression_ast = array_join_expression->toAST(options);
else
array_join_expression_ast = column_node->getExpression()->toAST(options);
}
else
array_join_expression_ast = array_join_expression->toAST(options);

View File

@ -22,6 +22,7 @@ public:
if (query_node->hasOrderBy())
{
QueryTreeNodeConstRawPtrWithHashSet unique_expressions_nodes_set;
QueryTreeNodes result_nodes;
auto & query_order_by_nodes = query_node->getOrderBy().getNodes();
@ -45,10 +46,9 @@ public:
query_order_by_nodes = std::move(result_nodes);
}
unique_expressions_nodes_set.clear();
if (query_node->hasLimitBy())
{
QueryTreeNodeConstRawPtrWithHashSet unique_expressions_nodes_set;
QueryTreeNodes result_nodes;
auto & query_limit_by_nodes = query_node->getLimitBy().getNodes();
@ -63,9 +63,6 @@ public:
query_limit_by_nodes = std::move(result_nodes);
}
}
private:
QueryTreeNodeConstRawPtrWithHashSet unique_expressions_nodes_set;
};
}

View File

@ -57,6 +57,7 @@ AllocationTrace CurrentMemoryTracker::allocImpl(Int64 size, bool throw_if_memory
{
auto res = memory_tracker->allocImpl(will_be, throw_if_memory_exceeded);
current_thread->untracked_memory = 0;
current_thread->updateUntrackedMemoryLimit(memory_tracker->get());
return res;
}
else
@ -84,6 +85,13 @@ void CurrentMemoryTracker::check()
std::ignore = memory_tracker->allocImpl(0, true);
}
Int64 CurrentMemoryTracker::get()
{
if (auto * memory_tracker = getMemoryTracker())
return memory_tracker->get();
return 0;
}
AllocationTrace CurrentMemoryTracker::alloc(Int64 size)
{
bool throw_if_memory_exceeded = true;
@ -103,10 +111,12 @@ AllocationTrace CurrentMemoryTracker::free(Int64 size)
if (current_thread)
{
current_thread->untracked_memory -= size;
if (current_thread->untracked_memory < -current_thread->untracked_memory_limit)
// Note that we use `max_untracked_memory` and not `untracked_memory_limit` to create hysteresis to avoid track/untrack cycles
if (current_thread->untracked_memory < -current_thread->max_untracked_memory)
{
Int64 untracked_memory = current_thread->untracked_memory;
current_thread->untracked_memory = 0;
current_thread->updateUntrackedMemoryLimit(memory_tracker->get() + untracked_memory);
return memory_tracker->free(-untracked_memory);
}
}

View File

@ -12,7 +12,9 @@ struct CurrentMemoryTracker
/// This function should be called after memory deallocation.
[[nodiscard]] static AllocationTrace free(Int64 size);
static void check();
[[nodiscard]] static Int64 get();
/// Throws MEMORY_LIMIT_EXCEEDED (if it's allowed to throw exceptions)
static void injectFault();

View File

@ -140,6 +140,18 @@ inline bool isPrintableASCII(char c)
return uc >= 32 && uc <= 126; /// 127 is ASCII DEL.
}
inline bool isCSIParameterByte(char c)
{
uint8_t uc = c;
return uc >= 0x30 && uc <= 0x3F; /// ASCII 09:;<=>?
}
inline bool isCSIIntermediateByte(char c)
{
uint8_t uc = c;
return uc >= 0x20 && uc <= 0x2F; /// ASCII !"#$%&'()*+,-./
}
inline bool isCSIFinalByte(char c)
{
uint8_t uc = c;

View File

@ -183,6 +183,12 @@ public:
Int64 untracked_memory = 0;
/// Each thread could new/delete memory in range of (-untracked_memory_limit, untracked_memory_limit) without access to common counters.
Int64 untracked_memory_limit = 4 * 1024 * 1024;
/// To keep total untracked memory limited to `untracked_memory_ratio * RSS` we have to account threads with small and large memory footprint differently.
/// For this purpose we dynamically change `untracked_memory_limit` after every tracking event using a simple formula:
/// untracked_memory_limit = clamp(untracked_memory_ratio * cur_memory_bytes, min_untracked_memory, max_untracked_memory)
/// Note that this values are updated when thread is attached to a group
Int64 min_untracked_memory = 4 * 1024 * 1024; // Default value is kept 4MB mostly for tests and client (should be changed to 4KB as default value a setting)
Int64 max_untracked_memory = 4 * 1024 * 1024;
/// Statistics of read and write rows/bytes
Progress progress_in;
@ -309,6 +315,12 @@ public:
void initGlobalProfiler(UInt64 global_profiler_real_time_period, UInt64 global_profiler_cpu_time_period);
void updateUntrackedMemoryLimit(Int64 current)
{
constexpr Int64 untracked_memory_ratio_bits = 4; // untracked_memory_ratio = 1.0 / (1 << untracked_memory_ratio_bits) = 1.0 / 16 = 6.25%
untracked_memory_limit = std::clamp<Int64>(current >> untracked_memory_ratio_bits, min_untracked_memory, max_untracked_memory);
}
private:
void applyGlobalSettings();
void applyQuerySettings();

View File

@ -103,7 +103,7 @@ template <ComputeWidthMode mode>
size_t computeWidthImpl(const UInt8 * data, size_t size, size_t prefix, size_t limit) noexcept
{
UTF8Decoder decoder;
int isEscapeSequence = false;
bool is_escape_sequence = false;
size_t width = 0;
size_t rollback = 0;
for (size_t i = 0; i < size; ++i)
@ -116,6 +116,9 @@ size_t computeWidthImpl(const UInt8 * data, size_t size, size_t prefix, size_t l
while (i + 15 < size)
{
if (is_escape_sequence)
break;
__m128i bytes = _mm_loadu_si128(reinterpret_cast<const __m128i *>(&data[i]));
const uint16_t non_regular_width_mask = _mm_movemask_epi8(
@ -132,25 +135,28 @@ size_t computeWidthImpl(const UInt8 * data, size_t size, size_t prefix, size_t l
}
else
{
if (isEscapeSequence)
{
break;
}
else
{
i += 16;
width += 16;
}
i += 16;
width += 16;
}
}
#endif
while (i < size && isPrintableASCII(data[i]))
{
if (!isEscapeSequence)
bool ignore_width = is_escape_sequence && (isCSIParameterByte(data[i]) || isCSIIntermediateByte(data[i]));
if (ignore_width || (data[i] == '[' && is_escape_sequence))
{
/// don't count the width
}
else if (is_escape_sequence && isCSIFinalByte(data[i]))
{
is_escape_sequence = false;
}
else
{
++width;
else if (isCSIFinalByte(data[i]) && data[i - 1] != '\x1b')
isEscapeSequence = false; /// end of CSI escape sequence reached
}
++i;
}
@ -178,7 +184,7 @@ size_t computeWidthImpl(const UInt8 * data, size_t size, size_t prefix, size_t l
// special treatment for '\t' and for ESC
size_t next_width = width;
if (decoder.codepoint == '\x1b')
isEscapeSequence = true;
is_escape_sequence = true;
else if (decoder.codepoint == '\t')
next_width += 8 - (prefix + width) % 8;
else

View File

@ -491,6 +491,7 @@ class IColumn;
M(UInt64, max_memory_usage_for_user, 0, "Maximum memory usage for processing all concurrently running queries for the user. Zero means unlimited.", 0) \
M(UInt64, memory_overcommit_ratio_denominator_for_user, 1_GiB, "It represents soft memory limit on the global level. This value is used to compute query overcommit ratio.", 0) \
M(UInt64, max_untracked_memory, (4 * 1024 * 1024), "Small allocations and deallocations are grouped in thread local variable and tracked or profiled only when amount (in absolute value) becomes larger than specified value. If the value is higher than 'memory_profiler_step' it will be effectively lowered to 'memory_profiler_step'.", 0) \
M(UInt64, min_untracked_memory, (4 * 1024), "Lower bound for untracked memory limit which is applied to threads with low memory consumption. Untracked memory limit equals thread_memory_usage/16 and clamped between min_untracked_memory and max_untracked_memory for every thread.", 0) \
M(UInt64, memory_profiler_step, (4 * 1024 * 1024), "Whenever query memory usage becomes larger than every next step in number of bytes the memory profiler will collect the allocating stack trace. Zero means disabled memory profiler. Values lower than a few megabytes will slow down query processing.", 0) \
M(Float, memory_profiler_sample_probability, 0., "Collect random allocations and deallocations and write them into system.trace_log with 'MemorySample' trace_type. The probability is for every alloc/free regardless to the size of the allocation (can be changed with `memory_profiler_sample_min_allocation_size` and `memory_profiler_sample_max_allocation_size`). Note that sampling happens only when the amount of untracked memory exceeds 'max_untracked_memory'. You may want to set 'max_untracked_memory' to 0 for extra fine grained sampling.", 0) \
M(UInt64, memory_profiler_sample_min_allocation_size, 0, "Collect random allocations of size greater or equal than specified value with probability equal to `memory_profiler_sample_probability`. 0 means disabled. You may want to set 'max_untracked_memory' to 0 to make this threshold to work as expected.", 0) \

View File

@ -96,6 +96,7 @@ static const std::map<ClickHouseVersion, SettingsChangesHistory::SettingsChanges
{"hdfs_ignore_file_doesnt_exist", false, false, "Allow to return 0 rows when the requested files don't exist instead of throwing an exception in HDFS table engine"},
{"azure_ignore_file_doesnt_exist", false, false, "Allow to return 0 rows when the requested files don't exist instead of throwing an exception in AzureBlobStorage table engine"},
{"s3_ignore_file_doesnt_exist", false, false, "Allow to return 0 rows when the requested files don't exist instead of throwing an exception in S3 table engine"},
{"min_untracked_memory", 4_MiB, 4_KiB, "A new setting to enable more accurate memory tracking."},
{"enable_blob_storage_log", true, true, "Write information about blob storage operations to system.blob_storage_log table"},
{"allow_statistic_optimize", false, false, "Old setting which popped up here being renamed."},
{"allow_experimental_statistic", false, false, "Old setting which popped up here being renamed."},

View File

@ -1,20 +1,21 @@
#include <filesystem>
#include <Databases/DatabaseAtomic.h>
#include <Databases/DatabaseFactory.h>
#include <Databases/DatabaseOnDisk.h>
#include <Databases/DatabaseReplicated.h>
#include <Databases/DatabaseFactory.h>
#include <IO/ReadBufferFromFile.h>
#include <IO/ReadHelpers.h>
#include <IO/WriteHelpers.h>
#include <IO/ReadBufferFromFile.h>
#include <Interpreters/Context.h>
#include <Interpreters/DDLTask.h>
#include <Interpreters/DatabaseCatalog.h>
#include <Interpreters/ExternalDictionariesLoader.h>
#include <Parsers/formatAST.h>
#include <Storages/StorageMaterializedView.h>
#include "Common/logger_useful.h"
#include <Common/PoolId.h>
#include <Common/atomicRename.h>
#include <Common/filesystemHelpers.h>
#include <Storages/StorageMaterializedView.h>
#include <Interpreters/Context.h>
#include <Interpreters/DatabaseCatalog.h>
#include <Interpreters/ExternalDictionariesLoader.h>
#include <filesystem>
#include <Interpreters/DDLTask.h>
namespace fs = std::filesystem;
@ -393,6 +394,7 @@ DatabaseAtomic::DetachedTables DatabaseAtomic::cleanupDetachedTables()
{
DetachedTables not_in_use;
auto it = detached_tables.begin();
LOG_DEBUG(log, "There are {} detached tables. Start searching non used tables.", detached_tables.size());
while (it != detached_tables.end())
{
if (it->second.unique())
@ -403,6 +405,7 @@ DatabaseAtomic::DetachedTables DatabaseAtomic::cleanupDetachedTables()
else
++it;
}
LOG_DEBUG(log, "Found {} non used tables in detached tables.", not_in_use.size());
/// It should be destroyed in caller with released database mutex
return not_in_use;
}

View File

@ -61,7 +61,7 @@ public:
return std::make_shared<DataTypeTuple>(tuple_arg_types);
}
ColumnPtr executeImpl(const ColumnsWithTypeAndName & arguments, const DataTypePtr &, size_t) const override
ColumnPtr executeImpl(const ColumnsWithTypeAndName & arguments, const DataTypePtr &, size_t input_rows_count) const override
{
const size_t num_arguments = arguments.size();
Columns columns;
@ -92,6 +92,9 @@ public:
columns.push_back(inner_col);
}
if (columns.empty())
return ColumnTuple::create(input_rows_count);
return ColumnTuple::create(columns);
}
};

View File

@ -596,6 +596,34 @@ void ActionsDAG::removeUnusedActions(const std::unordered_set<const Node *> & us
std::erase_if(inputs, [&](const Node * node) { return !visited_nodes.contains(node); });
}
void ActionsDAG::removeAliasesForFilter(const std::string & filter_name)
{
const auto & filter_node = findInOutputs(filter_name);
std::stack<Node *> stack;
stack.push(const_cast<Node *>(&filter_node));
std::unordered_set<const Node *> visited;
visited.insert(stack.top());
while (!stack.empty())
{
auto * node = stack.top();
stack.pop();
for (auto & child : node->children)
{
while (child->type == ActionType::ALIAS)
child = child->children.front();
if (!visited.contains(child))
{
stack.push(const_cast<Node *>(child));
visited.insert(child);
}
}
}
}
ActionsDAGPtr ActionsDAG::cloneSubDAG(const NodeRawConstPtrs & outputs, bool remove_aliases)
{
auto actions = std::make_shared<ActionsDAG>();
@ -1830,7 +1858,8 @@ ActionsDAG::SplitResult ActionsDAG::split(std::unordered_set<const Node *> split
input_node.result_name = child->result_name;
child_data.to_second = &second_nodes.emplace_back(std::move(input_node));
new_inputs.push_back(child);
if (child->type != ActionType::INPUT)
new_inputs.push_back(child);
}
}

View File

@ -195,6 +195,8 @@ public:
/// Remove actions that are not needed to compute output nodes with required names
void removeUnusedActions(const NameSet & required_names, bool allow_remove_inputs = true, bool allow_constant_folding = true);
void removeAliasesForFilter(const std::string & filter_name);
/// Transform the current DAG in a way that leaf nodes get folded into their parents. It's done
/// because each projection can provide some columns as inputs to substitute certain sub-DAGs
/// (expressions). Consider the following example:

View File

@ -11,6 +11,7 @@
#include <Parsers/formatAST.h>
#include <Parsers/queryNormalization.h>
#include <Common/CurrentThread.h>
#include <Common/CurrentMemoryTracker.h>
#include <Common/Exception.h>
#include <Common/ProfileEvents.h>
#include <Common/QueryProfiler.h>
@ -210,9 +211,12 @@ void ThreadStatus::applyQuerySettings()
query_id_from_query_context = query_context_ptr->getCurrentQueryId();
initQueryProfiler();
untracked_memory_limit = settings.max_untracked_memory;
if (settings.memory_profiler_step && settings.memory_profiler_step < static_cast<UInt64>(untracked_memory_limit))
untracked_memory_limit = settings.memory_profiler_step;
max_untracked_memory = settings.max_untracked_memory;
if (settings.memory_profiler_step && settings.memory_profiler_step < static_cast<UInt64>(max_untracked_memory))
max_untracked_memory = settings.memory_profiler_step;
min_untracked_memory = std::min<Int64>(settings.min_untracked_memory, max_untracked_memory);
updateUntrackedMemoryLimit(CurrentMemoryTracker::get());
#if defined(OS_LINUX)
/// Set "nice" value if required.

View File

@ -59,9 +59,6 @@ Token quotedStringWithUnicodeQuotes(const char *& pos, const char * const token_
pos = find_first_symbols<'\xE2'>(pos, end);
if (pos + 2 >= end)
return Token(error_token, token_begin, end);
/// Empty identifiers are not allowed, while empty strings are.
if (success_token == TokenType::QuotedIdentifier && pos + 3 >= end)
return Token(error_token, token_begin, end);
if (pos[0] == '\xE2' && pos[1] == '\x80' && pos[2] == expected_end_byte)
{

View File

@ -269,7 +269,12 @@ convertFieldToORCLiteral(const orc::Type & orc_type, const Field & field, DataTy
case orc::SHORT:
case orc::INT:
case orc::LONG: {
/// May throw exception
/// May throw exception.
///
/// In particular, it'll throw if we request the column as unsigned, like this:
/// SELECT * FROM file('t.orc', ORC, 'x UInt8') WHERE x > 10
/// We have to reject this, otherwise it would miss values > 127 (because
/// they're treated as negative by ORC).
auto val = field.get<Int64>();
return orc::Literal(val);
}

View File

@ -315,18 +315,20 @@ void ORCBlockOutputFormat::writeColumn(
if (null_bytemap)
orc_column.hasNulls = true;
/// ORC doesn't have unsigned types, so cast everything to signed and sign-extend to Int64 to
/// make the ORC library calculate min and max correctly.
switch (type->getTypeId())
{
case TypeIndex::Enum8: [[fallthrough]];
case TypeIndex::Int8:
{
/// Note: Explicit cast to avoid clang-tidy error: 'signed char' to 'long' conversion; consider casting to 'unsigned char' first.
writeNumbers<Int8, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const Int8 & value){ return static_cast<int64_t>(value); });
writeNumbers<Int8, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const Int8 & value){ return Int64(Int8(value)); });
break;
}
case TypeIndex::UInt8:
{
writeNumbers<UInt8, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const UInt8 & value){ return value; });
writeNumbers<UInt8, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const UInt8 & value){ return Int64(Int8(value)); });
break;
}
case TypeIndex::Enum16: [[fallthrough]];
@ -338,7 +340,7 @@ void ORCBlockOutputFormat::writeColumn(
case TypeIndex::Date: [[fallthrough]];
case TypeIndex::UInt16:
{
writeNumbers<UInt16, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const UInt16 & value){ return value; });
writeNumbers<UInt16, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const UInt16 & value){ return Int64(Int16(value)); });
break;
}
case TypeIndex::Date32: [[fallthrough]];
@ -349,12 +351,12 @@ void ORCBlockOutputFormat::writeColumn(
}
case TypeIndex::UInt32:
{
writeNumbers<UInt32, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const UInt32 & value){ return value; });
writeNumbers<UInt32, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const UInt32 & value){ return Int64(Int32(value)); });
break;
}
case TypeIndex::IPv4:
{
writeNumbers<IPv4, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const IPv4 & value){ return value.toUnderType(); });
writeNumbers<IPv4, orc::LongVectorBatch>(orc_column, column, null_bytemap, [](const IPv4 & value){ return Int64(Int32(value.toUnderType())); });
break;
}
case TypeIndex::Int64:

View File

@ -50,6 +50,8 @@ FilterStep::FilterStep(
, filter_column_name(std::move(filter_column_name_))
, remove_filter_column(remove_filter_column_)
{
actions_dag = actions_dag->clone();
actions_dag->removeAliasesForFilter(filter_column_name);
}
void FilterStep::transformPipeline(QueryPipelineBuilder & pipeline, const BuildQueryPipelineSettings & settings)

View File

@ -442,7 +442,11 @@ def _configure_jobs(
# filter jobs in accordance with ci settings
job_configs = ci_settings.apply(
job_configs, pr_info.is_release, is_pr=pr_info.is_pr, labels=pr_info.labels
job_configs,
pr_info.is_release,
is_pr=pr_info.is_pr,
is_mq=pr_info.is_merge_queue,
labels=pr_info.labels,
)
# check jobs in ci cache

View File

@ -134,6 +134,7 @@ class CiSettings:
job_config: JobConfig,
is_release: bool,
is_pr: bool,
is_mq: bool,
labels: Iterable[str],
) -> bool: # type: ignore #too-many-return-statements
if self.do_not_test:
@ -189,7 +190,7 @@ class CiSettings:
if job_config.release_only and not is_release:
return False
elif job_config.pr_only and not is_pr:
elif job_config.pr_only and not is_pr and not is_mq:
return False
return not to_deny
@ -199,6 +200,7 @@ class CiSettings:
job_configs: Dict[str, JobConfig],
is_release: bool,
is_pr: bool,
is_mq: bool,
labels: Iterable[str],
) -> Dict[str, JobConfig]:
"""
@ -207,16 +209,24 @@ class CiSettings:
res = {}
for job, job_config in job_configs.items():
if self._check_if_selected(
job, job_config, is_release=is_release, is_pr=is_pr, labels=labels
job,
job_config,
is_release=is_release,
is_pr=is_pr,
is_mq=is_mq,
labels=labels,
):
res[job] = job_config
add_parents = []
for job in list(res):
parent_jobs = CI_CONFIG.get_job_parents(job)
for parent_job in parent_jobs:
if parent_job not in res:
add_parents.append(parent_job)
print(f"Job [{job}] requires [{parent_job}] - add")
res[parent_job] = job_configs[parent_job]
for job in add_parents:
res[job] = job_configs[job]
for job, job_config in res.items():
batches = []

View File

@ -179,7 +179,11 @@ class TestCIOptions(unittest.TestCase):
)
filtered_jobs = list(
ci_options.apply(
jobs_configs, is_release=False, is_pr=True, labels=["TEST_LABEL"]
jobs_configs,
is_release=False,
is_pr=True,
is_mq=False,
labels=["TEST_LABEL"],
)
)
self.assertCountEqual(
@ -212,7 +216,9 @@ class TestCIOptions(unittest.TestCase):
jobs_configs["fuzzers"].run_by_label = "TEST_LABEL"
# no settings are set
filtered_jobs = list(
CiSettings().apply(jobs_configs, is_release=False, is_pr=True, labels=[])
CiSettings().apply(
jobs_configs, is_release=False, is_pr=False, is_mq=True, labels=[]
)
)
self.assertCountEqual(
filtered_jobs,
@ -220,9 +226,21 @@ class TestCIOptions(unittest.TestCase):
"Fast test",
],
)
filtered_jobs = list(
CiSettings().apply(jobs_configs, is_release=True, is_pr=False, labels=[])
CiSettings().apply(
jobs_configs, is_release=False, is_pr=True, is_mq=False, labels=[]
)
)
self.assertCountEqual(
filtered_jobs,
[
"Fast test",
],
)
filtered_jobs = list(
CiSettings().apply(
jobs_configs, is_release=True, is_pr=False, is_mq=False, labels=[]
)
)
self.assertCountEqual(
filtered_jobs,
@ -240,7 +258,11 @@ class TestCIOptions(unittest.TestCase):
# no settings are set
filtered_jobs = list(
ci_settings.apply(
jobs_configs, is_release=False, is_pr=True, labels=["TEST_LABEL"]
jobs_configs,
is_release=False,
is_pr=True,
is_mq=False,
labels=["TEST_LABEL"],
)
)
self.assertCountEqual(
@ -253,7 +275,11 @@ class TestCIOptions(unittest.TestCase):
ci_settings.include_keywords = ["Fast"]
filtered_jobs = list(
ci_settings.apply(
jobs_configs, is_release=True, is_pr=False, labels=["TEST_LABEL"]
jobs_configs,
is_release=True,
is_pr=False,
is_mq=False,
labels=["TEST_LABEL"],
)
)
self.assertCountEqual(
@ -277,7 +303,11 @@ class TestCIOptions(unittest.TestCase):
jobs_configs["Integration tests (asan)"].release_only = True
filtered_jobs = list(
ci_options.apply(
jobs_configs, is_release=False, is_pr=True, labels=["TEST_LABEL"]
jobs_configs,
is_release=False,
is_pr=True,
is_mq=False,
labels=["TEST_LABEL"],
)
)
self.assertCountEqual(

View File

@ -808,10 +808,10 @@ class SettingsRandomizer:
"merge_tree_coarse_index_granularity": lambda: random.randint(2, 32),
"optimize_distinct_in_order": lambda: random.randint(0, 1),
"max_bytes_before_external_sort": threshold_generator(
0.3, 0.5, 1, 10 * 1024 * 1024 * 1024
0.3, 0.5, 0, 10 * 1024 * 1024 * 1024
),
"max_bytes_before_external_group_by": threshold_generator(
0.3, 0.5, 1, 10 * 1024 * 1024 * 1024
0.3, 0.5, 0, 10 * 1024 * 1024 * 1024
),
"max_bytes_before_remerge_sort": lambda: random.randint(1, 3000000000),
"min_compress_block_size": lambda: random.randint(1, 1048576 * 3),
@ -850,6 +850,11 @@ class SettingsRandomizer:
"merge_tree_read_split_ranges_into_intersecting_and_non_intersecting_injection_probability": lambda: round(
random.random(), 2
),
"prefer_external_sort_block_bytes": lambda: random.choice([0, 1, 100000000]),
"cross_join_min_rows_to_compress": lambda: random.choice([0, 1, 100000000]),
"cross_join_min_bytes_to_compress": lambda: random.choice([0, 1, 100000000]),
"min_external_table_block_size_bytes": lambda: random.choice([0, 1, 100000000]),
"max_parsing_threads": lambda: random.choice([0, 1, 10]),
}
@staticmethod

View File

@ -110,10 +110,9 @@ class HDFSApi(object):
logging.debug(
"Stdout:\n{}\n".format(res.stdout.decode("utf-8"))
)
logging.debug("Env:\n{}\n".format(env))
raise Exception(
"Command {} return non-zero code {}: {}".format(
args, res.returncode, res.stderr.decode("utf-8")
cmd, res.returncode, res.stderr.decode("utf-8")
)
)

View File

@ -8,7 +8,7 @@ sys.path.insert(0, os.path.join(CURDIR))
from . import uexpect
prompt = ":\) "
prompt = ":\\) "
end_of_block = r".*\r\n.*\r\n"
@ -21,7 +21,7 @@ class client(object):
self.client.eol("\r")
self.client.logger(log, prefix=name)
self.client.timeout(20)
self.client.expect("[#\$] ", timeout=2)
self.client.expect("[#\\$] ", timeout=2)
self.client.send(command)
def __enter__(self):

View File

@ -1474,7 +1474,7 @@ def test_backup_all(exclude_system_log_tables):
restore_settings = []
if not exclude_system_log_tables:
restore_settings.append("allow_non_empty_tables=true")
restore_command = f"RESTORE ALL FROM {backup_name} {'SETTINGS '+ ', '.join(restore_settings) if restore_settings else ''}"
restore_command = f"RESTORE ALL FROM {backup_name} {'SETTINGS ' + ', '.join(restore_settings) if restore_settings else ''}"
session_id = new_session_id()
instance.http_query(

View File

@ -161,13 +161,13 @@ def wait_for_fail_restore(node, restore_id):
elif status == "RESTORING":
assert_eq_with_retry(
node,
f"SELECT status FROM system.backups WHERE id = '{backup_id}'",
f"SELECT status FROM system.backups WHERE id = '{restore_id}'",
"RESTORE_FAILED",
sleep_time=2,
retry_count=50,
)
error = node.query(
f"SELECT error FROM system.backups WHERE id == '{backup_id}'"
f"SELECT error FROM system.backups WHERE id == '{restore_id}'"
).rstrip("\n")
assert re.search(
"Cannot restore the table default.tbl because it already contains some data",

View File

@ -187,7 +187,7 @@ def check_convert_all_dbs_to_atomic():
# 6 tables, MVs contain 2 rows (inner tables does not match regexp)
assert "8\t{}\n".format(8 * len("atomic")) == node.query(
"SELECT count(), sum(n) FROM atomic.merge".format(db)
"SELECT count(), sum(n) FROM atomic.merge"
)
node.query("DETACH TABLE ordinary.detached PERMANENTLY")

View File

@ -89,7 +89,7 @@ def test_aggregate_states(start_cluster):
logging.info("Skipping %s", aggregate_function)
skipped += 1
continue
logging.exception("Failed %s", function)
logging.exception("Failed %s", aggregate_function)
failed += 1
continue

View File

@ -116,7 +116,7 @@ def test_usage(cluster, node_name):
(id Int32) ENGINE = MergeTree() ORDER BY id
SETTINGS storage_policy = 'web';
""".format(
i, uuids[i], i, i
i, uuids[i]
)
)
@ -338,7 +338,7 @@ def test_page_cache(cluster):
(id Int32) ENGINE = MergeTree() ORDER BY id
SETTINGS storage_policy = 'web';
""".format(
i, uuids[i], i, i
i, uuids[i]
)
)

View File

@ -45,9 +45,7 @@ def test_failed_async_inserts(started_cluster):
ignore_error=True,
)
select_query = (
"SELECT value FROM system.events WHERE event == 'FailedAsyncInsertQuery'"
)
select_query = "SELECT value FROM system.events WHERE event == 'FailedAsyncInsertQuery' SETTINGS min_untracked_memory = '4Mi'"
assert node.query(select_query) == "4\n"

View File

@ -90,7 +90,7 @@ def wait_until_fully_merged(node, table):
except:
return
raise Exception(f"There are still merges on-going after {retry} assignments")
raise Exception(f"There are still merges on-going after {i} assignments")
def test_jbod_balanced_merge(start_cluster):

View File

@ -91,7 +91,7 @@ def test_jdbc_insert(started_cluster):
"""
CREATE TABLE test.test_insert ENGINE = Memory AS
SELECT * FROM test.ClickHouseTable;
SELECT *
SELECT *
FROM jdbc('{0}?mutation', 'INSERT INTO test.test_insert VALUES({1}, ''{1}'', ''{1}'')');
""".format(
datasource, records
@ -115,7 +115,7 @@ def test_jdbc_update(started_cluster):
"""
CREATE TABLE test.test_update ENGINE = Memory AS
SELECT * FROM test.ClickHouseTable;
SELECT *
SELECT *
FROM jdbc(
'{}?mutation',
'SET mutations_sync = 1; ALTER TABLE test.test_update UPDATE Str=''{}'' WHERE Num = {} - 1;'
@ -145,7 +145,7 @@ def test_jdbc_delete(started_cluster):
"""
CREATE TABLE test.test_delete ENGINE = Memory AS
SELECT * FROM test.ClickHouseTable;
SELECT *
SELECT *
FROM jdbc(
'{}?mutation',
'SET mutations_sync = 1; ALTER TABLE test.test_delete DELETE WHERE Num < {} - 1;'
@ -158,7 +158,7 @@ def test_jdbc_delete(started_cluster):
expected = records - 1
actual = instance.query(
"SELECT Str FROM jdbc('{}', 'SELECT * FROM test.test_delete')".format(
datasource, records
datasource
)
)
assert int(actual) == expected, "expecting {} but got {}".format(expected, actual)

View File

@ -1,5 +1,5 @@
#!/usr/bin/env python3
##!/usr/bin/env python3
import pytest
from helpers.cluster import ClickHouseCluster
import helpers.keeper_utils as keeper_utils

View File

@ -1,6 +1,5 @@
#!/usr/bin/env python3
#!/usr/bin/env python3
import pytest
from helpers.cluster import ClickHouseCluster
import helpers.keeper_utils as keeper_utils

View File

@ -1,6 +1,5 @@
#!/usr/bin/env python3
#!/usr/bin/env python3
import pytest
from helpers.cluster import ClickHouseCluster
import random

View File

@ -537,10 +537,7 @@ def test_freeze_unfreeze(cluster):
def test_apply_new_settings(cluster):
node = cluster.instances[NODE_NAME]
create_table(node, TABLE_NAME)
config_path = os.path.join(
SCRIPT_DIR,
"./_gen/disk_storage_conf.xml".format(cluster.instances_dir_name),
)
config_path = os.path.join(SCRIPT_DIR, "./_gen/disk_storage_conf.xml")
azure_query(
node, f"INSERT INTO {TABLE_NAME} VALUES {generate_values('2020-01-03', 4096)}"

View File

@ -179,9 +179,7 @@ def test_different_data_types(started_cluster):
for i in range(10):
col = random.choice(["a", "b", "c"])
cursor.execute("UPDATE test_data_types SET {} = {};".format(col, i))
cursor.execute(
"""UPDATE test_data_types SET i = '2020-12-12';""".format(col, i)
)
cursor.execute("UPDATE test_data_types SET i = '2020-12-12';")
check_tables_are_synchronized(instance, "test_data_types", "id")
@ -452,7 +450,7 @@ def test_many_concurrent_queries(started_cluster):
# also change primary key value
print("try update primary key {}".format(thread_id))
cursor.execute(
"UPDATE {table}_{} SET key=key%100000+100000*{} WHERE key%{}=0".format(
"UPDATE {} SET key=key%100000+100000*{} WHERE key%{}=0".format(
table_name, i + 1, i + 1
)
)

View File

@ -28,7 +28,7 @@ def parse_response_line(line):
if line.startswith("#"):
return {}
match = re.match("^([a-zA-Z_:][a-zA-Z0-9_:]+)(\{.*\})? -?(\d)", line)
match = re.match(r"^([a-zA-Z_:][a-zA-Z0-9_:]+)(\{.*\})? -?(\d)", line)
assert match, line
name, _, val = match.groups()
return {name: int(val)}

View File

@ -6,6 +6,7 @@ import time
import threading
import pytest
from helpers.client import QueryRuntimeException
from helpers.cluster import ClickHouseCluster
cluster = ClickHouseCluster(__file__)

View File

@ -136,7 +136,10 @@ def test_select_clamps_settings():
)
assert (
distributed.query(query, settings={"max_memory_usage": 1})
distributed.query(
query,
settings={"max_memory_usage": 1, "min_untracked_memory": 4 * 1024 * 1024},
)
== "node1\tmax_memory_usage\t11111111\n"
"node1\treadonly\t0\n"
"node2\tmax_memory_usage\t0\n"

View File

@ -4,7 +4,7 @@ import os
import json
import helpers.client
from helpers.cluster import ClickHouseCluster
from helpers.cluster import ClickHouseCluster, ClickHouseInstance
from helpers.test_tools import TSV
from helpers.s3_tools import prepare_s3_bucket, upload_directory, get_file_contents

View File

@ -1,5 +1,5 @@
import helpers.client
from helpers.cluster import ClickHouseCluster
from helpers.cluster import ClickHouseCluster, ClickHouseInstance
from helpers.test_tools import TSV
import pyspark

View File

@ -702,7 +702,7 @@ def test_rabbitmq_sharding_between_queues_publish(rabbitmq_cluster):
assert (
int(result1) == messages_num * threads_num
), "ClickHouse lost some messages: {}".format(result)
), "ClickHouse lost some messages: {}".format(result1)
assert int(result2) == 10
@ -1516,7 +1516,7 @@ def test_rabbitmq_hash_exchange(rabbitmq_cluster):
assert (
int(result1) == messages_num * threads_num
), "ClickHouse lost some messages: {}".format(result)
), "ClickHouse lost some messages: {}".format(result1)
assert int(result2) == 4 * num_tables
@ -1966,7 +1966,7 @@ def test_rabbitmq_many_consumers_to_each_queue(rabbitmq_cluster):
assert (
int(result1) == messages_num * threads_num
), "ClickHouse lost some messages: {}".format(result)
), "ClickHouse lost some messages: {}".format(result1)
# 4 tables, 2 consumers for each table => 8 consumer tags
assert int(result2) == 8
@ -2427,9 +2427,7 @@ def test_rabbitmq_drop_table_properly(rabbitmq_cluster):
time.sleep(30)
try:
exists = channel.queue_declare(
callback, queue="rabbit_queue_drop", passive=True
)
exists = channel.queue_declare(queue="rabbit_queue_drop", passive=True)
except Exception as e:
exists = False
@ -3364,7 +3362,7 @@ def test_rabbitmq_flush_by_block_size(rabbitmq_cluster):
routing_key="",
body=json.dumps({"key": 0, "value": 0}),
)
except e:
except Exception as e:
logging.debug(f"Got error: {str(e)}")
produce_thread = threading.Thread(target=produce)
@ -3442,7 +3440,7 @@ def test_rabbitmq_flush_by_time(rabbitmq_cluster):
)
logging.debug("Produced a message")
time.sleep(0.8)
except e:
except Exception as e:
logging.debug(f"Got error: {str(e)}")
produce_thread = threading.Thread(target=produce)

View File

@ -1850,7 +1850,7 @@ class TestCancelBackgroundMoving:
config = inspect.cleandoc(
f"""
<clickhouse>
<max_local_write_bandwidth_for_server>{ 256 * 1024 }</max_local_write_bandwidth_for_server>
<max_local_write_bandwidth_for_server>{256 * 1024}</max_local_write_bandwidth_for_server>
</clickhouse>
"""
)

View File

@ -325,7 +325,7 @@ def optimize_with_retry(node, table_name, retry=20):
settings={"optimize_throw_if_noop": "1"},
)
break
except e:
except:
time.sleep(0.5)

View File

@ -4,10 +4,10 @@ CREATE TABLE nested (x UInt64, filter UInt8, n Nested(a UInt64)) ENGINE = MergeT
INSERT INTO nested SELECT number, number % 2, range(number % 10) FROM system.numbers LIMIT 100000;
ALTER TABLE nested ADD COLUMN n.b Array(UInt64);
SELECT DISTINCT n.b FROM nested PREWHERE filter;
SELECT DISTINCT n.b FROM nested PREWHERE filter ORDER BY ALL;
ALTER TABLE nested ADD COLUMN n.c Array(UInt64) DEFAULT arrayMap(x -> x * 2, n.a);
SELECT DISTINCT n.c FROM nested PREWHERE filter;
SELECT DISTINCT n.a, n.c FROM nested PREWHERE filter;
SELECT DISTINCT n.c FROM nested PREWHERE filter ORDER BY ALL;
SELECT DISTINCT n.a, n.c FROM nested PREWHERE filter ORDER BY ALL;
DROP TABLE nested;

View File

@ -7,7 +7,8 @@
-- sizeof(HLL) is (2^K * 6 / 8)
-- hence max_memory_usage for 100 rows = (96<<10)*100 = 9830400
SET use_uncompressed_cache = 0;
SET use_uncompressed_cache = 0;
SET min_untracked_memory = '4Mi';
-- HashTable for UInt32 (used until (1<<13) elements), hence 8192 elements
SELECT 'UInt32';

View File

@ -49,16 +49,16 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01056_window_view_proc_hop_watch.wv")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01056_window_view_proc_hop_watch.mt VALUES (1, now('US/Samoa') + 3)"
)
client1.expect("1" + end_of_block)
client1.expect("Progress: 1.00 rows.*\)")
client1.expect("Progress: 1.00 rows.*\\)")
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -47,7 +47,7 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH db_01059_event_hop_watch_strict_asc.wv")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO db_01059_event_hop_watch_strict_asc.mt VALUES (1, toDateTime('1990/01/01 12:00:00', 'US/Samoa'));"
)
@ -57,7 +57,7 @@ with client(name="client1>", log=log) as client1, client(
)
client2.expect("Ok.")
client1.expect("1*1990-01-01 12:00:02" + end_of_block)
client1.expect("Progress: 1.00 rows.*\)")
client1.expect("Progress: 1.00 rows.*\\)")
client2.send(
"INSERT INTO db_01059_event_hop_watch_strict_asc.mt VALUES (1, toDateTime('1990/01/01 12:00:10', 'US/Samoa'));"
@ -65,11 +65,11 @@ with client(name="client1>", log=log) as client1, client(
client2.expect("Ok.")
client1.expect("1*1990-01-01 12:00:06" + end_of_block)
client1.expect("1*1990-01-01 12:00:08" + end_of_block)
client1.expect("Progress: 3.00 rows.*\)")
client1.expect("Progress: 3.00 rows.*\\)")
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -49,7 +49,7 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01062_window_view_event_hop_watch_asc.wv")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01062_window_view_event_hop_watch_asc.mt VALUES (1, toDateTime('1990/01/01 12:00:00', 'US/Samoa'));"
)
@ -69,11 +69,11 @@ with client(name="client1>", log=log) as client1, client(
client2.expect(prompt)
client1.expect("1" + end_of_block)
client1.expect("2" + end_of_block)
client1.expect("Progress: 3.00 rows.*\)")
client1.expect("Progress: 3.00 rows.*\\)")
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -50,7 +50,7 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01065_window_view_event_hop_watch_bounded.wv")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01065_window_view_event_hop_watch_bounded.mt VALUES (1, '1990/01/01 12:00:00');"
)
@ -72,7 +72,7 @@ with client(name="client1>", log=log) as client1, client(
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -49,23 +49,23 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01069_window_view_proc_tumble_watch.wv")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01069_window_view_proc_tumble_watch.mt VALUES (1, now('US/Samoa') + 3)"
)
client2.expect("Ok.")
client1.expect("1" + end_of_block)
client1.expect("Progress: 1.00 rows.*\)")
client1.expect("Progress: 1.00 rows.*\\)")
client2.send(
"INSERT INTO 01069_window_view_proc_tumble_watch.mt VALUES (1, now('US/Samoa') + 3)"
)
client2.expect("Ok.")
client1.expect("1" + end_of_block)
client1.expect("Progress: 2.00 rows.*\)")
client1.expect("Progress: 2.00 rows.*\\)")
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -49,7 +49,7 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01070_window_view_watch_events.wv EVENTS")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01070_window_view_watch_events.mt VALUES (1, toDateTime('1990/01/01 12:00:00', 'US/Samoa'));"
)
@ -59,11 +59,11 @@ with client(name="client1>", log=log) as client1, client(
)
client2.expect("Ok.")
client1.expect("1990-01-01 12:00:05" + end_of_block)
client1.expect("Progress: 1.00 rows.*\)")
client1.expect("Progress: 1.00 rows.*\\)")
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -55,7 +55,7 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01078_window_view_alter_query_watch.wv")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01078_window_view_alter_query_watch.mt VALUES (1, toDateTime('1990/01/01 12:00:00', 'US/Samoa'));"
)
@ -65,7 +65,7 @@ with client(name="client1>", log=log) as client1, client(
)
client2.expect("Ok.")
client1.expect("1" + end_of_block)
client1.expect("Progress: 1.00 rows.*\)")
client1.expect("Progress: 1.00 rows.*\\)")
client2.send(
"ALTER TABLE 01078_window_view_alter_query_watch.wv MODIFY QUERY SELECT count(a) * 2 AS count, hopEnd(wid) AS w_end FROM 01078_window_view_alter_query_watch.mt GROUP BY hop(timestamp, INTERVAL '2' SECOND, INTERVAL '3' SECOND, 'US/Samoa') AS wid"
)
@ -75,7 +75,7 @@ with client(name="client1>", log=log) as client1, client(
client1.expect(prompt)
client3.send("WATCH 01078_window_view_alter_query_watch.wv")
client3.expect("Query id" + end_of_block)
client3.expect("Progress: 0.00 rows.*\)")
client3.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01078_window_view_alter_query_watch.mt VALUES (1, toDateTime('1990/01/01 12:00:06', 'US/Samoa'));"
)
@ -85,11 +85,11 @@ with client(name="client1>", log=log) as client1, client(
)
client2.expect("Ok.")
client3.expect("2" + end_of_block)
client3.expect("Progress: 1.00 rows.*\)")
client3.expect("Progress: 1.00 rows.*\\)")
# send Ctrl-C
client3.send("\x03", eol="")
match = client3.expect("(%s)|([#\$] )" % prompt)
match = client3.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client3.send(client3.command)
client3.expect(prompt)

View File

@ -49,7 +49,7 @@ with client(name="client1>", log=log) as client1, client(
client1.send("WATCH 01082_window_view_watch_limit.wv LIMIT 1")
client1.expect("Query id" + end_of_block)
client1.expect("Progress: 0.00 rows.*\)")
client1.expect("Progress: 0.00 rows.*\\)")
client2.send(
"INSERT INTO 01082_window_view_watch_limit.mt VALUES (1, '1990/01/01 12:00:00');"
)
@ -59,7 +59,7 @@ with client(name="client1>", log=log) as client1, client(
)
client2.expect("Ok.")
client1.expect("1" + end_of_block)
client1.expect("Progress: 1.00 rows.*\)")
client1.expect("Progress: 1.00 rows.*\\)")
client1.expect("1 row" + end_of_block)
client1.expect(prompt)

View File

@ -44,12 +44,12 @@ Filter
9 10 1
> one condition of filter should be pushed down after aggregating, other condition is aliased
Filter column
ALIAS notEquals(s, 4) :: 1 -> and(notEquals(y, 0), notEquals(s, 4))
ALIAS notEquals(s, 4) :: 4 -> and(notEquals(y, 0), notEquals(s, 4)) UInt8 : 2
Aggregating
Filter column: notEquals(y, 0)
> (analyzer) one condition of filter should be pushed down after aggregating, other condition is aliased
Filter column
ALIAS notEquals(__table1.s, 4_UInt8) :: 0 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 4_UInt8))
ALIAS notEquals(__table1.s, 4_UInt8) :: 1 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 4_UInt8))
Aggregating
Filter column: notEquals(__table1.y, 0_UInt8)
0 1
@ -63,12 +63,12 @@ Filter column: notEquals(__table1.y, 0_UInt8)
9 10
> one condition of filter should be pushed down after aggregating, other condition is casted
Filter column
FUNCTION and(minus(s, 4) :: 1, 1 :: 3) -> and(notEquals(y, 0), minus(s, 4)) UInt8 : 2
FUNCTION and(minus(s, 4) :: 5, 1 :: 3) -> and(notEquals(y, 0), minus(s, 4))
Aggregating
Filter column: notEquals(y, 0)
> (analyzer) one condition of filter should be pushed down after aggregating, other condition is casted
Filter column
FUNCTION and(minus(__table1.s, 4_UInt8) :: 0, 1 :: 3) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 4_UInt8)) UInt8 : 2
FUNCTION and(minus(__table1.s, 4_UInt8) :: 1, 1 :: 3) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 4_UInt8))
Aggregating
Filter column: notEquals(__table1.y, 0_UInt8)
0 1
@ -82,12 +82,12 @@ Filter column: notEquals(__table1.y, 0_UInt8)
9 10
> one condition of filter should be pushed down after aggregating, other two conditions are ANDed
Filter column
FUNCTION and(minus(s, 8) :: 1, minus(s, 4) :: 2) -> and(notEquals(y, 0), minus(s, 8), minus(s, 4))
FUNCTION and(minus(s, 8) :: 5, minus(s, 4) :: 2) -> and(notEquals(y, 0), minus(s, 8), minus(s, 4))
Aggregating
Filter column: notEquals(y, 0)
> (analyzer) one condition of filter should be pushed down after aggregating, other two conditions are ANDed
Filter column
FUNCTION and(minus(__table1.s, 8_UInt8) :: 0, minus(__table1.s, 4_UInt8) :: 2) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 8_UInt8), minus(__table1.s, 4_UInt8))
FUNCTION and(minus(__table1.s, 8_UInt8) :: 1, minus(__table1.s, 4_UInt8) :: 2) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 8_UInt8), minus(__table1.s, 4_UInt8))
Aggregating
Filter column: notEquals(__table1.y, 0_UInt8)
0 1
@ -100,12 +100,12 @@ Filter column: notEquals(__table1.y, 0_UInt8)
9 10
> two conditions of filter should be pushed down after aggregating and ANDed, one condition is aliased
Filter column
ALIAS notEquals(s, 8) :: 1 -> and(notEquals(y, 0), notEquals(s, 8), minus(y, 4))
ALIAS notEquals(s, 8) :: 4 -> and(notEquals(y, 0), notEquals(s, 8), minus(y, 4))
Aggregating
Filter column: and(notEquals(y, 0), minus(y, 4))
> (analyzer) two conditions of filter should be pushed down after aggregating and ANDed, one condition is aliased
Filter column
ALIAS notEquals(__table1.s, 8_UInt8) :: 0 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 8_UInt8), minus(__table1.y, 4_UInt8))
ALIAS notEquals(__table1.s, 8_UInt8) :: 1 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 8_UInt8), minus(__table1.y, 4_UInt8))
Aggregating
Filter column: and(notEquals(__table1.y, 0_UInt8), minus(__table1.y, 4_UInt8))
0 1

View File

@ -49,14 +49,14 @@ $CLICKHOUSE_CLIENT --allow_experimental_analyzer=0 -q "
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s != 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: notEquals(y, 0)\|ALIAS notEquals(s, 4) :: 1 -> and(notEquals(y, 0), notEquals(s, 4))"
grep -o "Aggregating\|Filter column\|Filter column: notEquals(y, 0)\|ALIAS notEquals(s, 4) :: 4 -> and(notEquals(y, 0), notEquals(s, 4)) UInt8 : 2"
echo "> (analyzer) one condition of filter should be pushed down after aggregating, other condition is aliased"
$CLICKHOUSE_CLIENT --allow_experimental_analyzer=1 -q "
explain actions = 1 select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s != 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: notEquals(__table1.y, 0_UInt8)\|ALIAS notEquals(__table1.s, 4_UInt8) :: 0 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 4_UInt8))"
grep -o "Aggregating\|Filter column\|Filter column: notEquals(__table1.y, 0_UInt8)\|ALIAS notEquals(__table1.s, 4_UInt8) :: 1 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 4_UInt8))"
$CLICKHOUSE_CLIENT -q "
select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
@ -69,14 +69,14 @@ $CLICKHOUSE_CLIENT --allow_experimental_analyzer=0 -q "
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s - 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: notEquals(y, 0)\|FUNCTION and(minus(s, 4) :: 1, 1 :: 3) -> and(notEquals(y, 0), minus(s, 4)) UInt8 : 2"
grep -o "Aggregating\|Filter column\|Filter column: notEquals(y, 0)\|FUNCTION and(minus(s, 4) :: 5, 1 :: 3) -> and(notEquals(y, 0), minus(s, 4))"
echo "> (analyzer) one condition of filter should be pushed down after aggregating, other condition is casted"
$CLICKHOUSE_CLIENT --allow_experimental_analyzer=1 -q "
explain actions = 1 select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s - 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: notEquals(__table1.y, 0_UInt8)\|FUNCTION and(minus(__table1.s, 4_UInt8) :: 0, 1 :: 3) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 4_UInt8)) UInt8 : 2"
grep -o "Aggregating\|Filter column\|Filter column: notEquals(__table1.y, 0_UInt8)\|FUNCTION and(minus(__table1.s, 4_UInt8) :: 1, 1 :: 3) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 4_UInt8))"
$CLICKHOUSE_CLIENT -q "
select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
@ -89,14 +89,14 @@ $CLICKHOUSE_CLIENT --allow_experimental_analyzer=0 --convert_query_to_cnf=0 -q "
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s - 8 and s - 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: notEquals(y, 0)\|FUNCTION and(minus(s, 8) :: 1, minus(s, 4) :: 2) -> and(notEquals(y, 0), minus(s, 8), minus(s, 4))"
grep -o "Aggregating\|Filter column\|Filter column: notEquals(y, 0)\|FUNCTION and(minus(s, 8) :: 5, minus(s, 4) :: 2) -> and(notEquals(y, 0), minus(s, 8), minus(s, 4))"
echo "> (analyzer) one condition of filter should be pushed down after aggregating, other two conditions are ANDed"
$CLICKHOUSE_CLIENT --allow_experimental_analyzer=1 --convert_query_to_cnf=0 -q "
explain actions = 1 select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s - 8 and s - 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: notEquals(__table1.y, 0_UInt8)\|FUNCTION and(minus(__table1.s, 8_UInt8) :: 0, minus(__table1.s, 4_UInt8) :: 2) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 8_UInt8), minus(__table1.s, 4_UInt8))"
grep -o "Aggregating\|Filter column\|Filter column: notEquals(__table1.y, 0_UInt8)\|FUNCTION and(minus(__table1.s, 8_UInt8) :: 1, minus(__table1.s, 4_UInt8) :: 2) -> and(notEquals(__table1.y, 0_UInt8), minus(__table1.s, 8_UInt8), minus(__table1.s, 4_UInt8))"
$CLICKHOUSE_CLIENT -q "
select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
@ -109,14 +109,14 @@ $CLICKHOUSE_CLIENT --allow_experimental_analyzer=0 --convert_query_to_cnf=0 -q "
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s != 8 and y - 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: and(notEquals(y, 0), minus(y, 4))\|ALIAS notEquals(s, 8) :: 1 -> and(notEquals(y, 0), notEquals(s, 8), minus(y, 4))"
grep -o "Aggregating\|Filter column\|Filter column: and(notEquals(y, 0), minus(y, 4))\|ALIAS notEquals(s, 8) :: 4 -> and(notEquals(y, 0), notEquals(s, 8), minus(y, 4))"
echo "> (analyzer) two conditions of filter should be pushed down after aggregating and ANDed, one condition is aliased"
$CLICKHOUSE_CLIENT --allow_experimental_analyzer=1 --convert_query_to_cnf=0 -q "
explain actions = 1 select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y
) where y != 0 and s != 8 and y - 4
settings enable_optimize_predicate_expression=0" |
grep -o "Aggregating\|Filter column\|Filter column: and(notEquals(__table1.y, 0_UInt8), minus(__table1.y, 4_UInt8))\|ALIAS notEquals(__table1.s, 8_UInt8) :: 0 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 8_UInt8), minus(__table1.y, 4_UInt8))"
grep -o "Aggregating\|Filter column\|Filter column: and(notEquals(__table1.y, 0_UInt8), minus(__table1.y, 4_UInt8))\|ALIAS notEquals(__table1.s, 8_UInt8) :: 1 -> and(notEquals(__table1.y, 0_UInt8), notEquals(__table1.s, 8_UInt8), minus(__table1.y, 4_UInt8))"
$CLICKHOUSE_CLIENT -q "
select s, y from (
select sum(x) as s, y from (select number as x, number + 1 as y from numbers(10)) group by y

View File

@ -7,6 +7,6 @@ CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
${CLICKHOUSE_CLIENT} -q "create table insert_big_json(a String, b String) engine=MergeTree() order by tuple()";
python3 -c "[print('{{\"a\":\"{}\", \"b\":\"{}\"'.format('clickhouse'* 1000000, 'dbms' * 1000000)) for i in range(10)]; [print('{{\"a\":\"{}\", \"b\":\"{}\"}}'.format('clickhouse'* 100000, 'dbms' * 100000)) for i in range(10)]" 2>/dev/null | ${CLICKHOUSE_CLIENT} --min_chunk_bytes_for_parallel_parsing=10485760 --max_threads=0 --input_format_parallel_parsing=1 --max_memory_usage=0 -q "insert into insert_big_json FORMAT JSONEachRow" 2>&1 | grep -q "min_chunk_bytes_for_parallel_parsing" && echo "Ok." || echo "FAIL" ||:
python3 -c "[print('{{\"a\":\"{}\", \"b\":\"{}\"'.format('clickhouse'* 1000000, 'dbms' * 1000000)) for i in range(10)]; [print('{{\"a\":\"{}\", \"b\":\"{}\"}}'.format('clickhouse'* 100000, 'dbms' * 100000)) for i in range(10)]" 2>/dev/null | ${CLICKHOUSE_CLIENT} --min_chunk_bytes_for_parallel_parsing=10485760 --max_threads=0 --input_format_parallel_parsing=1 --max_memory_usage=0 --max_parsing_threads=2 -q "insert into insert_big_json FORMAT JSONEachRow" 2>&1 | grep -q "min_chunk_bytes_for_parallel_parsing" && echo "Ok." || echo "FAIL" ||:
${CLICKHOUSE_CLIENT} -q "drop table insert_big_json"

View File

@ -15,6 +15,6 @@ log = None
with client(name="client1>", log=log) as client1:
client1.expect(prompt)
client1.send("SELECT number FROM numbers(1000) FORMAT Null")
client1.expect("Progress: 1\.00 thousand rows, 8\.00 KB .*" + end_of_block)
client1.expect("0 rows in set. Elapsed: [\\w]{1}\.[\\w]{3} sec.")
client1.expect("Progress: 1\\.00 thousand rows, 8\\.00 KB .*" + end_of_block)
client1.expect("0 rows in set. Elapsed: [\\w]{1}\\.[\\w]{3} sec.")
client1.expect("Peak memory usage: .*B" + end_of_block)

View File

@ -87,7 +87,7 @@ Sorting (Stream): a ASC, b ASC
Sorting (Stream): __table1.a ASC, __table1.b ASC
Sorting (Stream): __table1.a ASC, __table1.b ASC
Sorting (Stream): __table1.a ASC, __table1.b ASC
Sorting (Stream): __table1.a ASC, b ASC
Sorting (Stream): a ASC, b ASC
-- disabled, check that sorting description for ReadFromMergeTree match ORDER BY columns
Sorting (Stream): __table1.a ASC
Sorting (Stream): __table1.a ASC

View File

@ -32,12 +32,12 @@ with client(
)
client1.expect(prompt)
client1.send(f"INSERT INTO test.infile_progress FROM INFILE '{filename}'")
client1.expect("Progress: 5.00 rows, 10.00 B.*\)")
client1.expect("Progress: 5.00 rows, 10.00 B.*\\)")
client1.expect(prompt)
# send Ctrl-C
client1.send("\x03", eol="")
match = client1.expect("(%s)|([#\$] )" % prompt)
match = client1.expect("(%s)|([#\\$] )" % prompt)
if match.groups()[1]:
client1.send(client1.command)
client1.expect(prompt)

View File

@ -24,10 +24,10 @@ Positions: 3 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 0
FUNCTION equals(__table1.id : 3, 0_UInt8 :: 2) -> equals(__table1.id, 0_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 4
FUNCTION equals(id :: 0, 0_UInt8 :: 2) -> equals(__table1.id, 0_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table)
Header: id UInt64
value String

View File

@ -1,8 +1,4 @@
number Nullable(Int64)
u8 Nullable(Int8)
u16 Nullable(Int16)
u32 Nullable(Int32)
u64 Nullable(Int64)
i8 Nullable(Int8)
i16 Nullable(Int16)
i32 Nullable(Int32)
@ -22,34 +18,34 @@ d64 Nullable(Decimal(18, 10))
d128 Nullable(Decimal(38, 20))
-- Go over all types individually
-- { echoOn }
select count(), sum(number) from file('02892.orc') where indexHint(u8 in (10, 15, 250));
800 4229600
select count(1), min(u8), max(u8) from file('02892.orc') where u8 in (10, 15, 250);
66 10 15
select count(), sum(number) from file('02892.orc') where indexHint(i8 in (10, 15, -6));
1100 5744450
select count(1), min(i8), max(i8) from file('02892.orc') where i8 in (10, 15, -6);
99 -6 15
select count(), sum(number) from file('02892.orc') where indexHint(i8 between -3 and 2);
1000 4999500
select count(1), min(i8), max(i8) from file('02892.orc') where i8 between -3 and 2;
208 -3 2
select count(), sum(number) from file('02892.orc') where indexHint(u16 between 4000 and 61000 or u16 == 42);
1800 6479100
select count(1), min(u16), max(u16) from file('02892.orc') where u16 between 4000 and 61000 or u16 == 42;
select count(), sum(number) from file('02892.orc') where indexHint(i16 between 4000 and 61000 or i16 == 42);
1200 1099400
select count(1), min(i16), max(i16) from file('02892.orc') where i16 between 4000 and 61000 or i16 == 42;
1002 42 5000
select count(), sum(number) from file('02892.orc') where indexHint(i16 between -150 and 250);
500 2474750
select count(1), min(i16), max(i16) from file('02892.orc') where i16 between -150 and 250;
401 -150 250
select count(), sum(number) from file('02892.orc') where indexHint(u32 in (42, 4294966296));
200 999900
select count(1), min(u32), max(u32) from file('02892.orc') where u32 in (42, 4294966296);
1 42 42
select count(), sum(number) from file('02892.orc') where indexHint(i32 in (42, -1000));
200 1099900
select count(1), min(i32), max(i32) from file('02892.orc') where i32 in (42, -1000);
2 -1000 42
select count(), sum(number) from file('02892.orc') where indexHint(i32 between -150 and 250);
500 2474750
select count(1), min(i32), max(i32) from file('02892.orc') where i32 between -150 and 250;
401 -150 250
select count(), sum(number) from file('02892.orc') where indexHint(u64 in (42, 18446744073709550616));
100 494950
select count(1), min(u64), max(u64) from file('02892.orc') where u64 in (42, 18446744073709550616);
1 42 42
select count(), sum(number) from file('02892.orc') where indexHint(i64 in (42, -1000));
200 1099900
select count(1), min(i64), max(i64) from file('02892.orc') where i64 in (42, -1000);
2 -1000 42
select count(), sum(number) from file('02892.orc') where indexHint(i64 between -150 and 250);
500 2474750
select count(1), min(i64), max(i64) from file('02892.orc') where i64 between -150 and 250;
@ -111,21 +107,21 @@ select count(), sum(number) from file('02892.orc') where indexHint(0);
0 \N
select count(), min(number), max(number) from file('02892.orc') where indexHint(0);
0 \N \N
select count(), sum(number) from file('02892.orc') where indexHint(s like '99%' or u64 == 2000);
select count(), sum(number) from file('02892.orc') where indexHint(s like '99%' or i64 == 2000);
300 1204850
select count(), min(s), max(s) from file('02892.orc') where (s like '99%' or u64 == 2000);
select count(), min(s), max(s) from file('02892.orc') where (s like '99%' or i64 == 2000);
12 2000 999
select count(), sum(number) from file('02892.orc') where indexHint(s like 'z%');
0 \N
select count(), min(s), max(s) from file('02892.orc') where (s like 'z%');
0 \N \N
select count(), sum(number) from file('02892.orc') where indexHint(u8 == 10 or 1 == 1);
select count(), sum(number) from file('02892.orc') where indexHint(i8 == 10 or 1 == 1);
10000 49995000
select count(), min(u8), max(u8) from file('02892.orc') where (u8 == 10 or 1 == 1);
select count(), min(i8), max(i8) from file('02892.orc') where (i8 == 10 or 1 == 1);
10000 -128 127
select count(), sum(number) from file('02892.orc') where indexHint(u8 < 0);
select count(), sum(number) from file('02892.orc') where indexHint(i8 < 0);
5300 26042350
select count(), min(u8), max(u8) from file('02892.orc') where (u8 < 0);
select count(), min(i8), max(i8) from file('02892.orc') where (i8 < 0);
5001 -128 -1
-- { echoOn }
select count(), sum(number) from file('02892.orc') where indexHint(sometimes_null is NULL);

View File

@ -1,4 +1,4 @@
-- Tags: no-fasttest, no-parallel, no-cpu-aarch64
-- Tags: no-fasttest, no-parallel
set output_format_orc_string_as_string = 1;
set output_format_orc_row_index_stride = 100;
@ -16,15 +16,9 @@ SET session_timezone = 'UTC';
-- Try all the types.
insert into function file('02892.orc')
-- Use negative numbers to test sign extension for signed types and lack of sign extension for
-- unsigned types.
with 5000 - number as n
select
number,
intDiv(n, 11)::UInt8 as u8,
n::UInt16 u16,
n::UInt32 as u32,
n::UInt64 as u64,
intDiv(n, 11)::Int8 as i8,
n::Int16 i16,
n::Int32 as i32,
@ -50,26 +44,26 @@ desc file('02892.orc');
-- Go over all types individually
-- { echoOn }
select count(), sum(number) from file('02892.orc') where indexHint(u8 in (10, 15, 250));
select count(1), min(u8), max(u8) from file('02892.orc') where u8 in (10, 15, 250);
select count(), sum(number) from file('02892.orc') where indexHint(i8 in (10, 15, -6));
select count(1), min(i8), max(i8) from file('02892.orc') where i8 in (10, 15, -6);
select count(), sum(number) from file('02892.orc') where indexHint(i8 between -3 and 2);
select count(1), min(i8), max(i8) from file('02892.orc') where i8 between -3 and 2;
select count(), sum(number) from file('02892.orc') where indexHint(u16 between 4000 and 61000 or u16 == 42);
select count(1), min(u16), max(u16) from file('02892.orc') where u16 between 4000 and 61000 or u16 == 42;
select count(), sum(number) from file('02892.orc') where indexHint(i16 between 4000 and 61000 or i16 == 42);
select count(1), min(i16), max(i16) from file('02892.orc') where i16 between 4000 and 61000 or i16 == 42;
select count(), sum(number) from file('02892.orc') where indexHint(i16 between -150 and 250);
select count(1), min(i16), max(i16) from file('02892.orc') where i16 between -150 and 250;
select count(), sum(number) from file('02892.orc') where indexHint(u32 in (42, 4294966296));
select count(1), min(u32), max(u32) from file('02892.orc') where u32 in (42, 4294966296);
select count(), sum(number) from file('02892.orc') where indexHint(i32 in (42, -1000));
select count(1), min(i32), max(i32) from file('02892.orc') where i32 in (42, -1000);
select count(), sum(number) from file('02892.orc') where indexHint(i32 between -150 and 250);
select count(1), min(i32), max(i32) from file('02892.orc') where i32 between -150 and 250;
select count(), sum(number) from file('02892.orc') where indexHint(u64 in (42, 18446744073709550616));
select count(1), min(u64), max(u64) from file('02892.orc') where u64 in (42, 18446744073709550616);
select count(), sum(number) from file('02892.orc') where indexHint(i64 in (42, -1000));
select count(1), min(i64), max(i64) from file('02892.orc') where i64 in (42, -1000);
select count(), sum(number) from file('02892.orc') where indexHint(i64 between -150 and 250);
select count(1), min(i64), max(i64) from file('02892.orc') where i64 between -150 and 250;
@ -117,17 +111,17 @@ select count(1), min(d128), max(128) from file('02892.orc') where (d128 between
select count(), sum(number) from file('02892.orc') where indexHint(0);
select count(), min(number), max(number) from file('02892.orc') where indexHint(0);
select count(), sum(number) from file('02892.orc') where indexHint(s like '99%' or u64 == 2000);
select count(), min(s), max(s) from file('02892.orc') where (s like '99%' or u64 == 2000);
select count(), sum(number) from file('02892.orc') where indexHint(s like '99%' or i64 == 2000);
select count(), min(s), max(s) from file('02892.orc') where (s like '99%' or i64 == 2000);
select count(), sum(number) from file('02892.orc') where indexHint(s like 'z%');
select count(), min(s), max(s) from file('02892.orc') where (s like 'z%');
select count(), sum(number) from file('02892.orc') where indexHint(u8 == 10 or 1 == 1);
select count(), min(u8), max(u8) from file('02892.orc') where (u8 == 10 or 1 == 1);
select count(), sum(number) from file('02892.orc') where indexHint(i8 == 10 or 1 == 1);
select count(), min(i8), max(i8) from file('02892.orc') where (i8 == 10 or 1 == 1);
select count(), sum(number) from file('02892.orc') where indexHint(u8 < 0);
select count(), min(u8), max(u8) from file('02892.orc') where (u8 < 0);
select count(), sum(number) from file('02892.orc') where indexHint(i8 < 0);
select count(), min(i8), max(i8) from file('02892.orc') where (i8 < 0);
-- { echoOff }
-- Nullable and LowCardinality.

View File

@ -0,0 +1,290 @@
-- { echoOn }
-- inequality operation
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
--
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
--
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
-- BETWEEN
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
1 10 alpha 0 0
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
1 10 alpha 0 0
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
0 0 1 5 ALPHA
0 0 4 25 delta
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
0 0 1 5 ALPHA
0 0 4 25 delta
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
0 0 1 5 ALPHA
0 0 4 25 delta
1 10 alpha 0 0
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
0 0 1 5 ALPHA
0 0 4 25 delta
1 10 alpha 0 0
2 15 beta 2 10 beta
3 20 gamma 0 0
--
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
-- Stupid condition
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 2 10 beta
3 20 gamma 0 0
-- Window functions with stupid condition
SET join_algorithm='hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 LEFT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA 15 3
2 15 beta 2 10 beta 15 3
3 20 gamma 0 0 15 3
SET join_algorithm='grace_hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 LEFT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA 15 3
2 15 beta 2 10 beta 15 3
3 20 gamma 0 0 15 3
SET join_algorithm='hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 INNER JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA 12.5 3
2 15 beta 2 10 beta 12.5 3
SET join_algorithm='grace_hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 INNER JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
1 10 alpha 1 5 ALPHA 12.5 3
2 15 beta 2 10 beta 12.5 3
SET join_algorithm='hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 RIGHT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta 8.333333333333334 7
1 10 alpha 1 5 ALPHA 8.333333333333334 7
2 15 beta 2 10 beta 8.333333333333334 7
SET join_algorithm='grace_hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 RIGHT JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta 8.333333333333334 7
1 10 alpha 1 5 ALPHA 8.333333333333334 7
2 15 beta 2 10 beta 8.333333333333334 7
SET join_algorithm='hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 FULL JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta 11.25 7
1 10 alpha 1 5 ALPHA 11.25 7
2 15 beta 2 10 beta 11.25 7
3 20 gamma 0 0 11.25 7
SET join_algorithm='grace_hash';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 FULL JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
0 0 4 25 delta 11.25 7
1 10 alpha 1 5 ALPHA 11.25 7
2 15 beta 2 10 beta 11.25 7
3 20 gamma 0 0 11.25 7
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;

View File

@ -0,0 +1,82 @@
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;
CREATE TABLE t1 (
key UInt32,
a UInt32,
attr String
) ENGINE = MergeTree ORDER BY key;
CREATE TABLE t2 (
key UInt32,
a UInt32,
attr String
) ENGINE = MergeTree ORDER BY key;
INSERT INTO t1 (key, a, attr) VALUES (1, 10, 'alpha'), (2, 15, 'beta'), (3, 20, 'gamma');
INSERT INTO t2 (key, a, attr) VALUES (1, 5, 'ALPHA'), (2, 10, 'beta'), (4, 25, 'delta');
SET allow_experimental_analyzer=1;
SET allow_experimental_join_condition=1;
SET join_use_nulls=0;
-- { echoOn }
-- inequality operation
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
--
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND t1.a > t2.key AND t1.key + t2.a > 1 ORDER BY ALL;
{% endfor -%}
{% endfor -%}
--
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.key < t2.a OR t1.a % 2 = 0) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
-- BETWEEN
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t2.a BETWEEN 8 AND t1.a) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
--
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.a IN (SELECT a FROM t2 WHERE a = 10)) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
-- Stupid condition
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
-- Window functions with stupid condition
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.*, AVG(t1.a) OVER () AS avg_b, SUM(t2.key) OVER () AS sum_c FROM t1 {{ join_type }} JOIN t2 ON t1.key == t2.key AND (t1.a * length(t2.attr) / length(t1.attr) <> t2.a + t1.key - t2.key) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;

View File

@ -0,0 +1,90 @@
-- { echoOn }
-- Support for query lower
SET join_algorithm='hash';
SELECT * FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT * FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='hash';
SELECT * FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
SET join_algorithm='grace_hash';
SELECT * FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
1 10 alpha 1 5 ALPHA
SET join_algorithm='hash';
SELECT * FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
SET join_algorithm='grace_hash';
SELECT * FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
SET join_algorithm='hash';
SELECT * FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
SET join_algorithm='grace_hash';
SELECT * FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
0 0 2 10 beta
0 0 4 25 delta
1 10 alpha 1 5 ALPHA
2 15 beta 0 0
3 20 gamma 0 0
-- Subquery JOIN
SET join_algorithm='hash';
SELECT * FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) LEFT JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
1 10 alpha 1 5 ALPHA 1 100
2 15 beta 0 0 0 0
3 20 gamma 0 0 0 0
SET join_algorithm='grace_hash';
SELECT * FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) LEFT JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
1 10 alpha 1 5 ALPHA 1 100
2 15 beta 0 0 0 0
3 20 gamma 0 0 0 0
SET join_algorithm='hash';
SELECT * FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) INNER JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
1 10 alpha 1 5 ALPHA 1 100
SET join_algorithm='grace_hash';
SELECT * FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) INNER JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
1 10 alpha 1 5 ALPHA 1 100
SET join_algorithm='hash';
SELECT * FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) RIGHT JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
0 0 0 0 0 10
0 0 0 0 2 1000
1 10 alpha 1 5 ALPHA 1 100
SET join_algorithm='grace_hash';
SELECT * FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) RIGHT JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
0 0 0 0 0 10
0 0 0 0 2 1000
1 10 alpha 1 5 ALPHA 1 100
SET join_algorithm='hash';
SELECT * FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) FULL JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
0 0 0 0 0 10
0 0 0 0 2 1000
0 0 2 10 beta 0 0
0 0 4 25 delta 0 0
1 10 alpha 1 5 ALPHA 1 100
2 15 beta 0 0 0 0
3 20 gamma 0 0 0 0
SET join_algorithm='grace_hash';
SELECT * FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) FULL JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
0 0 0 0 0 10
0 0 0 0 2 1000
0 0 2 10 beta 0 0
0 0 4 25 delta 0 0
1 10 alpha 1 5 ALPHA 1 100
2 15 beta 0 0 0 0
3 20 gamma 0 0 0 0
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;

View File

@ -0,0 +1,44 @@
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;
CREATE TABLE t1 (
key UInt32,
a UInt32,
attr String
) ENGINE = MergeTree ORDER BY key;
CREATE TABLE t2 (
key UInt32,
a UInt32,
attr String
) ENGINE = MergeTree ORDER BY key;
INSERT INTO t1 (key, a, attr) VALUES (1, 10, 'alpha'), (2, 15, 'beta'), (3, 20, 'gamma');
INSERT INTO t2 (key, a, attr) VALUES (1, 5, 'ALPHA'), (2, 10, 'beta'), (4, 25, 'delta');
SET allow_experimental_analyzer=1;
SET allow_experimental_join_condition=1;
SET join_use_nulls=0;
-- { echoOn }
-- Support for query lower
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT * FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
-- Subquery JOIN
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT * FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.attr != t2.attr) {{ join_type }} JOIN (SELECT * FROM VALUES('key UInt64, a UInt64', (0, 10), (1, 100), (2, 1000))) t3 ON t1.key=t3.key AND t2.key=t3.key AND t3.a!=t1.a AND t3.a!=t2.a ORDER BY ALL;
{% endfor -%}
{% endfor -%}
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;

View File

@ -0,0 +1,163 @@
-- { echoOn }
-- These queries work
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
2 15 2 10
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
2 15 2 10
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
2 15 2 10
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
2 15 2 10
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 0 0
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 0 0
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 2 10
0 0 4 25
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 2 10
0 0 4 25
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 2 10
0 0 4 25
1 10 0 0
2 15 0 0
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 2 10
0 0 4 25
1 10 0 0
2 15 0 0
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
2 15 2 10
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
2 15 2 10
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
2 15 2 10
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
2 15 2 10
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 LEFT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
2 15 2 10
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 INNER JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
2 15 2 10
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
2 15 2 10
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 RIGHT JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
2 15 2 10
SET join_algorithm='hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
1 10 0 0
2 15 2 10
3 20 0 0
SET join_algorithm='grace_hash';
SELECT t1.*, t2.* FROM t1 FULL JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
0 0 1 5
0 0 4 25
1 10 0 0
2 15 2 10
3 20 0 0

View File

@ -0,0 +1,51 @@
DROP TABLE IF EXISTS t1;
DROP TABLE IF EXISTS t2;
CREATE TABLE t1 (
key UInt32,
a UInt32
) ENGINE = MergeTree ORDER BY key;
CREATE TABLE t2 (
key UInt32,
a UInt32
) ENGINE = MergeTree ORDER BY key;
INSERT INTO t1 (key, a) VALUES (1, 10), (2, 15), (3, 20);
INSERT INTO t2 (key, a) VALUES (1, 5), (2, 10), (4, 25);
SET allow_experimental_analyzer=1;
SET allow_experimental_join_condition=1;
SET join_algorithm='hash';
-- { echoOn }
-- These queries work
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND ((t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.a=2 AND (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a = (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
{% endfor -%}
{% endfor -%}
{% for join_type in ['LEFT', 'INNER', 'RIGHT', 'FULL'] -%}
{% for algorithm in ['hash', 'grace_hash'] -%}
SET join_algorithm='{{ algorithm }}';
SELECT t1.*, t2.* FROM t1 {{ join_type }} JOIN t2 ON t1.key = t2.key AND (t1.a=2 OR (t2.a IN (SELECT a FROM t1 WHERE a = 10))) ORDER BY ALL;
{% endfor -%}
{% endfor -%}

View File

@ -13,6 +13,8 @@ create table dist_out as data engine=Distributed(test_shard_localhost, currentDa
set prefer_localhost_replica=0;
set min_untracked_memory='4Mi'; -- Disable precise memory tracking
insert into dist_in select number/100, number from system.numbers limit 1e6 settings max_memory_usage='20Mi';
system flush distributed dist_in; -- { serverError MEMORY_LIMIT_EXCEEDED }
system flush distributed dist_in settings max_memory_usage=0;

View File

@ -33,10 +33,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 0
FUNCTION equals(__table1.id : 3, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_1)
Header: id UInt64
value String
@ -50,10 +50,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 0
FUNCTION equals(__table2.id : 3, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_2)
Header: id UInt64
value String
@ -100,10 +100,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 0
FUNCTION equals(__table1.id : 3, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_1)
Header: id UInt64
value String
@ -117,10 +117,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 0
FUNCTION equals(__table2.id : 3, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_2)
Header: id UInt64
value String
@ -168,12 +168,12 @@ Positions: 4 2 0 1
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 6_UInt8 UInt8 : 2
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 3
ALIAS id :: 0 -> __table1.id UInt64 : 4
ALIAS value :: 1 -> __table1.value String : 0
FUNCTION equals(__table1.id : 4, 6_UInt8 :: 2) -> equals(__table1.id, 6_UInt8) UInt8 : 1
FUNCTION equals(__table1.id : 4, 5_UInt8 :: 3) -> equals(__table1.id, 5_UInt8) UInt8 : 2
ALIAS id : 0 -> __table1.id UInt64 : 4
ALIAS value :: 1 -> __table1.value String : 5
FUNCTION equals(id : 0, 6_UInt8 :: 2) -> equals(__table1.id, 6_UInt8) UInt8 : 1
FUNCTION equals(id :: 0, 5_UInt8 :: 3) -> equals(__table1.id, 5_UInt8) UInt8 : 2
FUNCTION and(equals(__table1.id, 5_UInt8) :: 2, equals(__table1.id, 6_UInt8) :: 1) -> and(equals(__table1.id, 5_UInt8), equals(__table1.id, 6_UInt8)) UInt8 : 3
Positions: 3 4 0
Positions: 3 4 5
ReadFromMergeTree (default.test_table_1)
Header: id UInt64
value String
@ -188,12 +188,12 @@ Positions: 4 2 0 1
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
COLUMN Const(UInt8) -> 6_UInt8 UInt8 : 3
ALIAS id :: 0 -> __table2.id UInt64 : 4
ALIAS value :: 1 -> __table2.value String : 0
FUNCTION equals(__table2.id : 4, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
FUNCTION equals(__table2.id : 4, 6_UInt8 :: 3) -> equals(__table2.id, 6_UInt8) UInt8 : 2
ALIAS id : 0 -> __table2.id UInt64 : 4
ALIAS value :: 1 -> __table2.value String : 5
FUNCTION equals(id : 0, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
FUNCTION equals(id :: 0, 6_UInt8 :: 3) -> equals(__table2.id, 6_UInt8) UInt8 : 2
FUNCTION and(equals(__table2.id, 6_UInt8) :: 2, equals(__table2.id, 5_UInt8) :: 1) -> and(equals(__table2.id, 6_UInt8), equals(__table2.id, 5_UInt8)) UInt8 : 3
Positions: 3 4 0
Positions: 3 4 5
ReadFromMergeTree (default.test_table_2)
Header: id UInt64
value String
@ -237,10 +237,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 0
FUNCTION equals(__table1.id : 3, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_1)
Header: id UInt64
value String
@ -254,10 +254,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 0
FUNCTION equals(__table2.id : 3, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_2)
Header: id UInt64
value String
@ -452,10 +452,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 0
FUNCTION equals(__table1.id : 3, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table1.id UInt64 : 3
ALIAS value :: 1 -> __table1.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table1.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_1)
Header: id UInt64
value String
@ -469,10 +469,10 @@ Positions: 4 2 0 1
Actions: INPUT : 0 -> id UInt64 : 0
INPUT : 1 -> value String : 1
COLUMN Const(UInt8) -> 5_UInt8 UInt8 : 2
ALIAS id :: 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 0
FUNCTION equals(__table2.id : 3, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 0
ALIAS id : 0 -> __table2.id UInt64 : 3
ALIAS value :: 1 -> __table2.value String : 4
FUNCTION equals(id :: 0, 5_UInt8 :: 2) -> equals(__table2.id, 5_UInt8) UInt8 : 1
Positions: 1 3 4
ReadFromMergeTree (default.test_table_2)
Header: id UInt64
value String

View File

@ -25,13 +25,12 @@ Positions: 4 0 2 1
Header: __table1.id UInt64
__table1.value String
Actions: INPUT : 1 -> value String : 0
INPUT :: 0 -> __table1.id UInt64 : 1
INPUT :: 2 -> id UInt64 : 2
ALIAS value :: 0 -> __table1.value String : 3
Positions: 1 3
INPUT : 0 -> id UInt64 : 1
ALIAS value :: 0 -> __table1.value String : 2
ALIAS id :: 1 -> __table1.id UInt64 : 0
Positions: 0 2
ReadFromMergeTree (default.test_table_1)
Header: __table1.id UInt64
id UInt64
Header: id UInt64
value String
ReadType: Default
Parts: 1
@ -42,20 +41,18 @@ Positions: 4 0 2 1
Prewhere filter column: notEquals(__table1.id, 0_UInt8) (removed)
Actions: INPUT : 0 -> id UInt64 : 0
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 1
ALIAS id : 0 -> __table1.id UInt64 : 2
FUNCTION notEquals(__table1.id : 2, 0_UInt8 :: 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 3
Positions: 2 0 3
FUNCTION notEquals(id : 0, 0_UInt8 :: 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 2
Positions: 0 2
Expression
Header: __table2.id UInt64
__table2.value String
Actions: INPUT : 1 -> value String : 0
INPUT :: 0 -> __table2.id UInt64 : 1
INPUT :: 2 -> id UInt64 : 2
ALIAS value :: 0 -> __table2.value String : 3
Positions: 1 3
INPUT : 0 -> id UInt64 : 1
ALIAS value :: 0 -> __table2.value String : 2
ALIAS id :: 1 -> __table2.id UInt64 : 0
Positions: 0 2
ReadFromMergeTree (default.test_table_2)
Header: __table2.id UInt64
id UInt64
Header: id UInt64
value String
ReadType: Default
Parts: 1
@ -66,9 +63,8 @@ Positions: 4 0 2 1
Prewhere filter column: notEquals(__table2.id, 0_UInt8) (removed)
Actions: INPUT : 0 -> id UInt64 : 0
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 1
ALIAS id : 0 -> __table2.id UInt64 : 2
FUNCTION notEquals(__table2.id : 2, 0_UInt8 :: 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 3
Positions: 2 0 3
FUNCTION notEquals(id : 0, 0_UInt8 :: 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 2
Positions: 0 2
--
2 Value_2 2 Value_2
--
@ -99,13 +95,12 @@ Positions: 4 0 2 1
Header: __table1.id UInt64
__table1.value String
Actions: INPUT : 1 -> value String : 0
INPUT :: 0 -> __table1.id UInt64 : 1
INPUT :: 2 -> id UInt64 : 2
ALIAS value :: 0 -> __table1.value String : 3
Positions: 1 3
INPUT : 0 -> id UInt64 : 1
ALIAS value :: 0 -> __table1.value String : 2
ALIAS id :: 1 -> __table1.id UInt64 : 0
Positions: 0 2
ReadFromMergeTree (default.test_table_1)
Header: __table1.id UInt64
id UInt64
Header: id UInt64
value String
ReadType: Default
Parts: 1
@ -116,20 +111,18 @@ Positions: 4 0 2 1
Prewhere filter column: notEquals(__table1.id, 0_UInt8) (removed)
Actions: INPUT : 0 -> id UInt64 : 0
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 1
ALIAS id : 0 -> __table1.id UInt64 : 2
FUNCTION notEquals(__table1.id : 2, 0_UInt8 :: 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 3
Positions: 2 0 3
FUNCTION notEquals(id : 0, 0_UInt8 :: 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 2
Positions: 0 2
Expression
Header: __table2.id UInt64
__table2.value String
Actions: INPUT : 1 -> value String : 0
INPUT :: 0 -> __table2.id UInt64 : 1
INPUT :: 2 -> id UInt64 : 2
ALIAS value :: 0 -> __table2.value String : 3
Positions: 1 3
INPUT : 0 -> id UInt64 : 1
ALIAS value :: 0 -> __table2.value String : 2
ALIAS id :: 1 -> __table2.id UInt64 : 0
Positions: 0 2
ReadFromMergeTree (default.test_table_2)
Header: __table2.id UInt64
id UInt64
Header: id UInt64
value String
ReadType: Default
Parts: 1
@ -140,9 +133,8 @@ Positions: 4 0 2 1
Prewhere filter column: notEquals(__table2.id, 0_UInt8) (removed)
Actions: INPUT : 0 -> id UInt64 : 0
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 1
ALIAS id : 0 -> __table2.id UInt64 : 2
FUNCTION notEquals(__table2.id : 2, 0_UInt8 :: 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 3
Positions: 2 0 3
FUNCTION notEquals(id : 0, 0_UInt8 :: 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 2
Positions: 0 2
--
2 Value_2 2 Value_2
--
@ -173,13 +165,12 @@ Positions: 4 0 2 1
Header: __table1.id UInt64
__table1.value String
Actions: INPUT : 1 -> value String : 0
INPUT :: 0 -> __table1.id UInt64 : 1
INPUT :: 2 -> id UInt64 : 2
ALIAS value :: 0 -> __table1.value String : 3
Positions: 1 3
INPUT : 0 -> id UInt64 : 1
ALIAS value :: 0 -> __table1.value String : 2
ALIAS id :: 1 -> __table1.id UInt64 : 0
Positions: 0 2
ReadFromMergeTree (default.test_table_1)
Header: __table1.id UInt64
id UInt64
Header: id UInt64
value String
ReadType: Default
Parts: 1
@ -190,22 +181,20 @@ Positions: 4 0 2 1
Prewhere filter column: and(notEquals(__table1.id, 0_UInt8), notEquals(__table1.id, 0_UInt8)) (removed)
Actions: INPUT : 0 -> id UInt64 : 0
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 1
ALIAS id : 0 -> __table1.id UInt64 : 2
FUNCTION notEquals(__table1.id : 2, 0_UInt8 : 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 3
FUNCTION notEquals(__table1.id : 2, 0_UInt8 :: 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 4
FUNCTION and(notEquals(__table1.id, 0_UInt8) :: 4, notEquals(__table1.id, 0_UInt8) :: 3) -> and(notEquals(__table1.id, 0_UInt8), notEquals(__table1.id, 0_UInt8)) UInt8 : 1
Positions: 2 0 1
FUNCTION notEquals(id : 0, 0_UInt8 : 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 2
FUNCTION notEquals(id : 0, 0_UInt8 :: 1) -> notEquals(__table1.id, 0_UInt8) UInt8 : 3
FUNCTION and(notEquals(__table1.id, 0_UInt8) :: 3, notEquals(__table1.id, 0_UInt8) :: 2) -> and(notEquals(__table1.id, 0_UInt8), notEquals(__table1.id, 0_UInt8)) UInt8 : 1
Positions: 0 1
Expression
Header: __table2.id UInt64
__table2.value String
Actions: INPUT : 1 -> value String : 0
INPUT :: 0 -> __table2.id UInt64 : 1
INPUT :: 2 -> id UInt64 : 2
ALIAS value :: 0 -> __table2.value String : 3
Positions: 1 3
INPUT : 0 -> id UInt64 : 1
ALIAS value :: 0 -> __table2.value String : 2
ALIAS id :: 1 -> __table2.id UInt64 : 0
Positions: 0 2
ReadFromMergeTree (default.test_table_2)
Header: __table2.id UInt64
id UInt64
Header: id UInt64
value String
ReadType: Default
Parts: 1
@ -216,10 +205,9 @@ Positions: 4 0 2 1
Prewhere filter column: and(notEquals(__table2.id, 0_UInt8), notEquals(__table2.id, 0_UInt8)) (removed)
Actions: INPUT : 0 -> id UInt64 : 0
COLUMN Const(UInt8) -> 0_UInt8 UInt8 : 1
ALIAS id : 0 -> __table2.id UInt64 : 2
FUNCTION notEquals(__table2.id : 2, 0_UInt8 : 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 3
FUNCTION notEquals(__table2.id : 2, 0_UInt8 :: 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 4
FUNCTION and(notEquals(__table2.id, 0_UInt8) :: 4, notEquals(__table2.id, 0_UInt8) :: 3) -> and(notEquals(__table2.id, 0_UInt8), notEquals(__table2.id, 0_UInt8)) UInt8 : 1
Positions: 2 0 1
FUNCTION notEquals(id : 0, 0_UInt8 : 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 2
FUNCTION notEquals(id : 0, 0_UInt8 :: 1) -> notEquals(__table2.id, 0_UInt8) UInt8 : 3
FUNCTION and(notEquals(__table2.id, 0_UInt8) :: 3, notEquals(__table2.id, 0_UInt8) :: 2) -> and(notEquals(__table2.id, 0_UInt8), notEquals(__table2.id, 0_UInt8)) UInt8 : 1
Positions: 0 1
--
2 Value_2 2 Value_2

View File

@ -1,5 +1,10 @@
┏━━━┓
┃ x ┃
┡━━━┩
1. │ █ │
1. │ █ │
└───┘
┏━━━━━━━━━┳━━━━━━━━━━┓
┃ 'Hello' ┃ x ┃
┡━━━━━━━━━╇━━━━━━━━━━┩
1. │ Hello │ █ test █ │
└─────────┴──────────┘

View File

@ -1 +1,2 @@
SELECT format('\x1b[38;2;{0};{1};{2}m█\x1b[0m', 255, 128, 0) AS x FORMAT Pretty;
SELECT format('\x1b[38;2;{0};{1};{2}m█\x1b[0m', 255, 128, 128) AS x FORMAT Pretty;
SELECT 'Hello', format('\x1b[38;2;{0};{1};{2}m█\x1b[0m test \x1b[38;2;{0};{1};{2}m█\x1b[0m', 255, 128, 128) AS x FORMAT Pretty;

View File

@ -10,3 +10,5 @@ Hello 1
Hello 1
Hello 2
Hello 2
2020-01-01 a 2
2020-01-01 b 4

View File

@ -8,3 +8,21 @@ SELECT s, arr, a FROM remote('127.0.0.{1,2}', currentDatabase(), arrays_test) AR
SELECT s, arr FROM remote('127.0.0.2', currentDatabase(), arrays_test) ARRAY JOIN arr WHERE arr < 3 ORDER BY arr;
SELECT s, arr FROM remote('127.0.0.{1,2}', currentDatabase(), arrays_test) ARRAY JOIN arr WHERE arr < 3 ORDER BY arr;
create table hourly(
hour datetime,
`metric.names` Array(String),
`metric.values` Array(Int64)
) Engine=Memory
as select '2020-01-01', ['a', 'b'], [1,2];
SELECT
toDate(hour) AS day,
`metric.names`,
sum(`metric.values`)
FROM remote('127.0.0.{1,2}', currentDatabase(), hourly)
ARRAY JOIN metric
GROUP BY
day,
metric.names
ORDER BY metric.names;

View File

@ -0,0 +1,41 @@
-- { echoOn }
select x from file('i8.orc') where indexHint(x = -128);
-128
select x from file('i8.orc') where indexHint(x = 128);
select x from file('u8.orc') where indexHint(x = -128);
-128
select x from file('u8.orc') where indexHint(x = 128);
select x from file('i16.orc') where indexHint(x = -32768);
-32768
select x from file('i16.orc') where indexHint(x = 32768);
select x from file('u16.orc') where indexHint(x = -32768);
-32768
select x from file('u16.orc') where indexHint(x = 32768);
select x from file('i32.orc') where indexHint(x = -2147483648);
-2147483648
select x from file('i32.orc') where indexHint(x = 2147483648);
select x from file('u32.orc') where indexHint(x = -2147483648);
-2147483648
select x from file('u32.orc') where indexHint(x = 2147483648);
select x from file('i64.orc') where indexHint(x = -9223372036854775808);
-9223372036854775808
select x from file('i64.orc') where indexHint(x = 9223372036854775808);
-9223372036854775808
select x from file('u64.orc') where indexHint(x = -9223372036854775808);
-9223372036854775808
select x from file('u64.orc') where indexHint(x = 9223372036854775808);
-9223372036854775808
select x from file('u8.orc', ORC, 'x UInt8') where indexHint(x > 10);
128
select x from file('u8.orc', ORC, 'x UInt64') where indexHint(x > 10);
18446744073709551488
select x from file('u16.orc', ORC, 'x UInt16') where indexHint(x > 10);
32768
select x from file('u16.orc', ORC, 'x UInt64') where indexHint(x > 10);
18446744073709518848
select x from file('u32.orc', ORC, 'x UInt32') where indexHint(x > 10);
2147483648
select x from file('u32.orc', ORC, 'x UInt64') where indexHint(x > 10);
18446744071562067968
select x from file('u64.orc', ORC, 'x UInt64') where indexHint(x > 10);
9223372036854775808

View File

@ -0,0 +1,42 @@
-- Tags: no-fasttest, no-parallel
set input_format_orc_filter_push_down = 1;
set engine_file_truncate_on_insert = 1;
insert into function file('i8.orc') select materialize(-128)::Int8 as x;
insert into function file('u8.orc') select materialize(128)::UInt8 as x;
insert into function file('i16.orc') select materialize(-32768)::Int16 as x;
insert into function file('u16.orc') select materialize(32768)::UInt16 as x;
insert into function file('i32.orc') select materialize(-2147483648)::Int32 as x;
insert into function file('u32.orc') select materialize(2147483648)::UInt32 as x;
insert into function file('i64.orc') select materialize(-9223372036854775808)::Int64 as x;
insert into function file('u64.orc') select materialize(9223372036854775808)::UInt64 as x;
-- { echoOn }
select x from file('i8.orc') where indexHint(x = -128);
select x from file('i8.orc') where indexHint(x = 128);
select x from file('u8.orc') where indexHint(x = -128);
select x from file('u8.orc') where indexHint(x = 128);
select x from file('i16.orc') where indexHint(x = -32768);
select x from file('i16.orc') where indexHint(x = 32768);
select x from file('u16.orc') where indexHint(x = -32768);
select x from file('u16.orc') where indexHint(x = 32768);
select x from file('i32.orc') where indexHint(x = -2147483648);
select x from file('i32.orc') where indexHint(x = 2147483648);
select x from file('u32.orc') where indexHint(x = -2147483648);
select x from file('u32.orc') where indexHint(x = 2147483648);
select x from file('i64.orc') where indexHint(x = -9223372036854775808);
select x from file('i64.orc') where indexHint(x = 9223372036854775808);
select x from file('u64.orc') where indexHint(x = -9223372036854775808);
select x from file('u64.orc') where indexHint(x = 9223372036854775808);
select x from file('u8.orc', ORC, 'x UInt8') where indexHint(x > 10);
select x from file('u8.orc', ORC, 'x UInt64') where indexHint(x > 10);
select x from file('u16.orc', ORC, 'x UInt16') where indexHint(x > 10);
select x from file('u16.orc', ORC, 'x UInt64') where indexHint(x > 10);
select x from file('u32.orc', ORC, 'x UInt32') where indexHint(x > 10);
select x from file('u32.orc', ORC, 'x UInt64') where indexHint(x > 10);
select x from file('u64.orc', ORC, 'x UInt64') where indexHint(x > 10);

View File

@ -0,0 +1,39 @@
QUERY id: 0
PROJECTION COLUMNS
id UInt64
PROJECTION
LIST id: 1, nodes: 1
COLUMN id: 2, column_name: id, result_type: UInt64, source_id: 3
JOIN TREE
TABLE id: 3, alias: __table1, table_name: default.test, final: 1
WHERE
FUNCTION id: 4, function_name: in, function_type: ordinary, result_type: UInt8
ARGUMENTS
LIST id: 5, nodes: 2
COLUMN id: 2, column_name: id, result_type: UInt64, source_id: 3
QUERY id: 6, is_subquery: 1, is_distinct: 1
PROJECTION COLUMNS
id UInt64
PROJECTION
LIST id: 7, nodes: 1
COLUMN id: 8, column_name: id, result_type: UInt64, source_id: 9
JOIN TREE
TABLE id: 9, alias: __table1, table_name: default.test, final: 1
ORDER BY
LIST id: 10, nodes: 1
SORT id: 11, sort_direction: ASCENDING, with_fill: 0
EXPRESSION
COLUMN id: 8, column_name: id, result_type: UInt64, source_id: 9
LIMIT
CONSTANT id: 12, constant_value: UInt64_4, constant_value_type: UInt64
ORDER BY
LIST id: 13, nodes: 1
SORT id: 14, sort_direction: ASCENDING, with_fill: 0
EXPRESSION
COLUMN id: 2, column_name: id, result_type: UInt64, source_id: 3
LIMIT BY LIMIT
CONSTANT id: 15, constant_value: UInt64_1, constant_value_type: UInt64
LIMIT BY
LIST id: 16, nodes: 1
COLUMN id: 2, column_name: id, result_type: UInt64, source_id: 3
SETTINGS allow_experimental_analyzer=1

View File

@ -0,0 +1,16 @@
CREATE TABLE test
ENGINE = ReplacingMergeTree
PRIMARY KEY id
AS SELECT number AS id FROM numbers(100);
EXPLAIN QUERY TREE SELECT id
FROM test FINAL
WHERE id IN (
SELECT DISTINCT id
FROM test FINAL
ORDER BY id ASC
LIMIT 4
)
ORDER BY id ASC
LIMIT 1 BY id
SETTINGS allow_experimental_analyzer = 1;

View File

@ -2,6 +2,8 @@ SELECT '-------- Bloom filter --------';
SELECT '';
DROP TABLE IF EXISTS 03165_token_bf;
SET allow_experimental_full_text_index=1;
CREATE TABLE 03165_token_bf
(
id Int64,

View File

@ -0,0 +1 @@
()

View File

@ -0,0 +1 @@
SELECT ()||();

View File

@ -0,0 +1 @@
test

View File

@ -0,0 +1 @@
SELECT test AS column

View File

@ -8,7 +8,7 @@ sys.path.insert(0, os.path.join(CURDIR))
import uexpect
prompt = ":\) "
prompt = ":\\) "
end_of_block = r".*\r\n.*\r\n"
@ -21,7 +21,7 @@ class client(object):
self.client.eol("\r")
self.client.logger(log, prefix=name)
self.client.timeout(120)
self.client.expect("[#\$] ", timeout=60)
self.client.expect("[#\\$] ", timeout=60)
self.client.send(command)
def __enter__(self):

View File

@ -10,7 +10,7 @@ import uexpect
class shell(object):
def __init__(self, command=None, name="", log=None, prompt="[#\$] "):
def __init__(self, command=None, name="", log=None, prompt="[#\\$] "):
if command is None:
command = ["/bin/bash", "--noediting"]
self.prompt = prompt

View File

@ -8,6 +8,7 @@ import shutil
import zipfile # For reading backups from zip archives
import boto3 # For reading backups from S3
import botocore
## Examples:

55
utils/check-style/check-flake8 Executable file
View File

@ -0,0 +1,55 @@
#!/usr/bin/env bash
function join_by() { local IFS="$1"; shift; echo "$*"; }
set -e
# We check only our code, that's why we skip contrib
GIT_ROOT=$(git rev-parse --show-cdup)
GIT_ROOT=${GIT_ROOT:-./}
# Find all *.py, *.python files and executable files without extension
# that are determined as python scripts by 'file' util
# in the repo except the contrib directory.
find_cmd=(
find "$GIT_ROOT" -type f -not -path "${GIT_ROOT}contrib/*"
\(
\(
-name '*.py' -or -name "*.python" -or
\(
-executable -not -name "*.*" -exec sh -c 'file {} | grep -q "Python script"' \;
\)
\)
# We skip modules generated by the protocol buffer compiler from *.proto files.
-and -not -name '*_pb2.py' -and -not -name '*_pb2_grpc.py'
\) -print0
)
ignores=(
E101 # Indentation contains mixed spaces and tabs
E203 # Whitespace before ':'
E226 # missing whitespace around arithmetic operator
E266 # Too many leading '#' for block comment
E401 # Multiple imports on one line
E402 # Module level import not at top of file
E501 # line too long
E711 # Comparison to None should be 'cond is None:'
E712 # Comparison to true should be 'if cond is true:' or 'if cond:'
E713 # Test for membership should be 'not in'
E714 # Test for object identity should be 'is not'
E722 # Do not use bare except, specify exception instead
E731 # Do not assign a lambda expression, use a def
E741 # Do not use variables named 'I', 'O', or 'l'
F401 # Module imported but unused
F403 # 'from module import *' used; unable to detect undefined names
F405 # Name may be undefined, or defined from star imports: module
F522 # .format(...) unused named arguments
F541 # f-string without any placeholders
F811 # redefinition of unused name from line N
F841 # local variable name is assigned to but never used
W191 # Indentation contains tabs
W291 # Trailing whitespace
W293 # Blank line contains whitespace
W503 # Line break occurred before a binary operator
)
"${find_cmd[@]}" | xargs -0 flake8 --ignore "$(join_by , "${ignores[@]}")"

View File

@ -18,6 +18,7 @@ def process_result(result_folder):
"style",
"pylint",
"black",
"flake8",
"mypy",
"typos",
"whitespaces",