mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-22 23:52:03 +00:00
Merge remote-tracking branch 'blessed/master' into allow_experimental_join_condition
This commit is contained in:
commit
2f42ace740
2
contrib/cld2
vendored
2
contrib/cld2
vendored
@ -1 +1 @@
|
||||
Subproject commit bc6d493a2f64ed1fc1c4c4b4294a542a04e04217
|
||||
Subproject commit 217ba8b8805b41557faadaa47bb6e99f2242eea3
|
@ -54,6 +54,7 @@ SELECT * FROM test_table;
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## See also
|
||||
|
||||
|
@ -235,6 +235,7 @@ libhdfs3 support HDFS namenode HA.
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Storage Settings {#storage-settings}
|
||||
|
||||
|
@ -53,14 +53,14 @@ For partitioning by month, use the `toYYYYMM(date_column)` expression, where `da
|
||||
|
||||
This example uses the [docker compose recipe](https://github.com/ClickHouse/examples/tree/5fdc6ff72f4e5137e23ea075c88d3f44b0202490/docker-compose-recipes/recipes/ch-and-minio-S3), which integrates ClickHouse and MinIO. You should be able to reproduce the same queries using S3 by replacing the endpoint and authentication values.
|
||||
|
||||
Notice that the S3 endpoint in the `ENGINE` configuration uses the parameter token `{_partition_id}` as part of the S3 object (filename), and that the SELECT queries select against those resulting object names (e.g., `test_3.csv`).
|
||||
Notice that the S3 endpoint in the `ENGINE` configuration uses the parameter token `{_partition_id}` as part of the S3 object (filename), and that the SELECT queries select against those resulting object names (e.g., `test_3.csv`).
|
||||
|
||||
:::note
|
||||
As shown in the example, querying from S3 tables that are partitioned is
|
||||
not directly supported at this time, but can be accomplished by querying the individual partitions
|
||||
using the S3 table function.
|
||||
|
||||
The primary use-case for writing
|
||||
The primary use-case for writing
|
||||
partitioned data in S3 is to enable transferring that data into another
|
||||
ClickHouse system (for example, moving from on-prem systems to ClickHouse
|
||||
Cloud). Because ClickHouse datasets are often very large, and network
|
||||
@ -78,9 +78,9 @@ CREATE TABLE p
|
||||
)
|
||||
ENGINE = S3(
|
||||
# highlight-next-line
|
||||
'http://minio:10000/clickhouse//test_{_partition_id}.csv',
|
||||
'minioadmin',
|
||||
'minioadminpassword',
|
||||
'http://minio:10000/clickhouse//test_{_partition_id}.csv',
|
||||
'minioadmin',
|
||||
'minioadminpassword',
|
||||
'CSV')
|
||||
PARTITION BY column3
|
||||
```
|
||||
@ -145,6 +145,7 @@ Code: 48. DB::Exception: Received from localhost:9000. DB::Exception: Reading fr
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
For more information about virtual columns see [here](../../../engines/table-engines/index.md#table_engines-virtual_columns).
|
||||
|
||||
|
@ -102,6 +102,7 @@ For partitioning by month, use the `toYYYYMM(date_column)` expression, where `da
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Settings {#settings}
|
||||
|
||||
|
@ -108,6 +108,7 @@ For partitioning by month, use the `toYYYYMM(date_column)` expression, where `da
|
||||
- `_path` — Path to the `URL`. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Resource name of the `URL`. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the resource in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Storage Settings {#storage-settings}
|
||||
|
||||
|
@ -72,6 +72,7 @@ SELECT count(*) FROM azureBlobStorage('DefaultEndpointsProtocol=https;AccountNam
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the file size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
**See Also**
|
||||
|
||||
|
@ -196,6 +196,7 @@ SELECT count(*) FROM file('big_dir/**/file002', 'CSV', 'name String, value UInt3
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the file size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Settings {#settings}
|
||||
|
||||
|
@ -97,6 +97,7 @@ FROM hdfs('hdfs://hdfs1:9000/big_dir/file{0..9}{0..9}{0..9}', 'CSV', 'name Strin
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Storage Settings {#storage-settings}
|
||||
|
||||
|
@ -272,6 +272,7 @@ FROM s3(
|
||||
- `_path` — Path to the file. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Name of the file. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the file in bytes. Type: `Nullable(UInt64)`. If the file size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Storage Settings {#storage-settings}
|
||||
|
||||
|
@ -53,6 +53,7 @@ Character `|` inside patterns is used to specify failover addresses. They are it
|
||||
- `_path` — Path to the `URL`. Type: `LowCardinalty(String)`.
|
||||
- `_file` — Resource name of the `URL`. Type: `LowCardinalty(String)`.
|
||||
- `_size` — Size of the resource in bytes. Type: `Nullable(UInt64)`. If the size is unknown, the value is `NULL`.
|
||||
- `_time` — Last modified time of the file. Type: `Nullable(DateTime)`. If the time is unknown, the value is `NULL`.
|
||||
|
||||
## Storage Settings {#storage-settings}
|
||||
|
||||
|
@ -773,7 +773,27 @@ try
|
||||
LOG_INFO(log, "Available CPU instruction sets: {}", cpu_info);
|
||||
#endif
|
||||
|
||||
bool will_have_trace_collector = hasPHDRCache() && config().has("trace_log");
|
||||
bool has_trace_collector = false;
|
||||
/// Disable it if we collect test coverage information, because it will work extremely slow.
|
||||
#if !WITH_COVERAGE
|
||||
/// Profilers cannot work reliably with any other libunwind or without PHDR cache.
|
||||
has_trace_collector = hasPHDRCache() && config().has("trace_log");
|
||||
#endif
|
||||
|
||||
/// Describe multiple reasons when query profiler cannot work.
|
||||
|
||||
#if WITH_COVERAGE
|
||||
LOG_INFO(log, "Query Profiler and TraceCollector are disabled because they work extremely slow with test coverage.");
|
||||
#endif
|
||||
|
||||
#if defined(SANITIZER)
|
||||
LOG_INFO(log, "Query Profiler disabled because they cannot work under sanitizers"
|
||||
" when two different stack unwinding methods will interfere with each other.");
|
||||
#endif
|
||||
|
||||
if (!hasPHDRCache())
|
||||
LOG_INFO(log, "Query Profiler and TraceCollector are disabled because they require PHDR cache to be created"
|
||||
" (otherwise the function 'dl_iterate_phdr' is not lock free and not async-signal safe).");
|
||||
|
||||
// Initialize global thread pool. Do it before we fetch configs from zookeeper
|
||||
// nodes (`from_zk`), because ZooKeeper interface uses the pool. We will
|
||||
@ -782,8 +802,27 @@ try
|
||||
server_settings.max_thread_pool_size,
|
||||
server_settings.max_thread_pool_free_size,
|
||||
server_settings.thread_pool_queue_size,
|
||||
will_have_trace_collector ? server_settings.global_profiler_real_time_period_ns : 0,
|
||||
will_have_trace_collector ? server_settings.global_profiler_cpu_time_period_ns : 0);
|
||||
has_trace_collector ? server_settings.global_profiler_real_time_period_ns : 0,
|
||||
has_trace_collector ? server_settings.global_profiler_cpu_time_period_ns : 0);
|
||||
|
||||
if (has_trace_collector)
|
||||
{
|
||||
global_context->createTraceCollector();
|
||||
|
||||
/// Set up server-wide memory profiler (for total memory tracker).
|
||||
if (server_settings.total_memory_profiler_step)
|
||||
total_memory_tracker.setProfilerStep(server_settings.total_memory_profiler_step);
|
||||
|
||||
if (server_settings.total_memory_tracker_sample_probability > 0.0)
|
||||
total_memory_tracker.setSampleProbability(server_settings.total_memory_tracker_sample_probability);
|
||||
|
||||
if (server_settings.total_memory_profiler_sample_min_allocation_size)
|
||||
total_memory_tracker.setSampleMinAllocationSize(server_settings.total_memory_profiler_sample_min_allocation_size);
|
||||
|
||||
if (server_settings.total_memory_profiler_sample_max_allocation_size)
|
||||
total_memory_tracker.setSampleMaxAllocationSize(server_settings.total_memory_profiler_sample_max_allocation_size);
|
||||
}
|
||||
|
||||
/// Wait for all threads to avoid possible use-after-free (for example logging objects can be already destroyed).
|
||||
SCOPE_EXIT({
|
||||
Stopwatch watch;
|
||||
@ -1950,52 +1989,9 @@ try
|
||||
|
||||
LOG_DEBUG(log, "Loaded metadata.");
|
||||
|
||||
/// Init trace collector only after trace_log system table was created
|
||||
/// Disable it if we collect test coverage information, because it will work extremely slow.
|
||||
#if !WITH_COVERAGE
|
||||
/// Profilers cannot work reliably with any other libunwind or without PHDR cache.
|
||||
if (hasPHDRCache())
|
||||
{
|
||||
if (has_trace_collector)
|
||||
global_context->initializeTraceCollector();
|
||||
|
||||
/// Set up server-wide memory profiler (for total memory tracker).
|
||||
if (server_settings.total_memory_profiler_step)
|
||||
{
|
||||
total_memory_tracker.setProfilerStep(server_settings.total_memory_profiler_step);
|
||||
}
|
||||
|
||||
if (server_settings.total_memory_tracker_sample_probability > 0.0)
|
||||
{
|
||||
total_memory_tracker.setSampleProbability(server_settings.total_memory_tracker_sample_probability);
|
||||
}
|
||||
|
||||
if (server_settings.total_memory_profiler_sample_min_allocation_size)
|
||||
{
|
||||
total_memory_tracker.setSampleMinAllocationSize(server_settings.total_memory_profiler_sample_min_allocation_size);
|
||||
}
|
||||
|
||||
if (server_settings.total_memory_profiler_sample_max_allocation_size)
|
||||
{
|
||||
total_memory_tracker.setSampleMaxAllocationSize(server_settings.total_memory_profiler_sample_max_allocation_size);
|
||||
}
|
||||
}
|
||||
#endif
|
||||
|
||||
/// Describe multiple reasons when query profiler cannot work.
|
||||
|
||||
#if WITH_COVERAGE
|
||||
LOG_INFO(log, "Query Profiler and TraceCollector are disabled because they work extremely slow with test coverage.");
|
||||
#endif
|
||||
|
||||
#if defined(SANITIZER)
|
||||
LOG_INFO(log, "Query Profiler disabled because they cannot work under sanitizers"
|
||||
" when two different stack unwinding methods will interfere with each other.");
|
||||
#endif
|
||||
|
||||
if (!hasPHDRCache())
|
||||
LOG_INFO(log, "Query Profiler and TraceCollector are disabled because they require PHDR cache to be created"
|
||||
" (otherwise the function 'dl_iterate_phdr' is not lock free and not async-signal safe).");
|
||||
|
||||
#if defined(OS_LINUX)
|
||||
auto tasks_stats_provider = TasksStatsCounters::findBestAvailableProvider();
|
||||
if (tasks_stats_provider == TasksStatsCounters::MetricsProvider::None)
|
||||
|
@ -289,10 +289,14 @@ void executeColumnIfNeeded(ColumnWithTypeAndName & column, bool empty)
|
||||
if (!column_function)
|
||||
return;
|
||||
|
||||
size_t original_size = column.column->size();
|
||||
|
||||
if (!empty)
|
||||
column = column_function->reduce();
|
||||
else
|
||||
column.column = column_function->getResultType()->createColumn();
|
||||
column.column = column_function->getResultType()->createColumnConstWithDefaultValue(original_size)->convertToFullColumnIfConst();
|
||||
|
||||
chassert(column.column->size() == original_size);
|
||||
}
|
||||
|
||||
int checkShortCircuitArguments(const ColumnsWithTypeAndName & arguments)
|
||||
|
@ -228,9 +228,9 @@ void Timer::cleanup()
|
||||
#endif
|
||||
|
||||
template <typename ProfilerImpl>
|
||||
QueryProfilerBase<ProfilerImpl>::QueryProfilerBase([[maybe_unused]] UInt64 thread_id, [[maybe_unused]] int clock_type, [[maybe_unused]] UInt32 period, [[maybe_unused]] int pause_signal_)
|
||||
: log(getLogger("QueryProfiler"))
|
||||
, pause_signal(pause_signal_)
|
||||
QueryProfilerBase<ProfilerImpl>::QueryProfilerBase(
|
||||
[[maybe_unused]] UInt64 thread_id, [[maybe_unused]] int clock_type, [[maybe_unused]] UInt32 period, [[maybe_unused]] int pause_signal_)
|
||||
: log(getLogger("QueryProfiler")), pause_signal(pause_signal_)
|
||||
{
|
||||
#if defined(SANITIZER)
|
||||
throw Exception(ErrorCodes::NOT_IMPLEMENTED, "QueryProfiler disabled because they cannot work under sanitizers");
|
||||
|
@ -5,6 +5,7 @@
|
||||
#include <functional>
|
||||
#include <memory>
|
||||
|
||||
#include <Poco/Timestamp.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
@ -25,6 +26,7 @@ public:
|
||||
{
|
||||
UInt64 uncompressed_size;
|
||||
UInt64 compressed_size;
|
||||
Poco::Timestamp last_modified;
|
||||
bool is_encrypted;
|
||||
};
|
||||
|
||||
|
@ -157,6 +157,7 @@ public:
|
||||
file_info.emplace();
|
||||
file_info->uncompressed_size = archive_entry_size(current_entry);
|
||||
file_info->compressed_size = archive_entry_size(current_entry);
|
||||
file_info->last_modified = archive_entry_mtime(current_entry);
|
||||
file_info->is_encrypted = false;
|
||||
}
|
||||
|
||||
|
@ -740,12 +740,18 @@ struct ContextSharedPart : boost::noncopyable
|
||||
|
||||
void initializeTraceCollector(std::shared_ptr<TraceLog> trace_log)
|
||||
{
|
||||
if (!trace_log)
|
||||
return;
|
||||
if (!trace_collector.has_value())
|
||||
throw Exception(ErrorCodes::LOGICAL_ERROR, "TraceCollector needs to be first created before initialization");
|
||||
|
||||
trace_collector->initialize(trace_log);
|
||||
}
|
||||
|
||||
void createTraceCollector()
|
||||
{
|
||||
if (hasTraceCollector())
|
||||
return;
|
||||
|
||||
trace_collector.emplace(std::move(trace_log));
|
||||
trace_collector.emplace();
|
||||
}
|
||||
|
||||
void addWarningMessage(const String & message) TSA_REQUIRES(mutex)
|
||||
@ -3891,6 +3897,11 @@ void Context::initializeSystemLogs()
|
||||
});
|
||||
}
|
||||
|
||||
void Context::createTraceCollector()
|
||||
{
|
||||
shared->createTraceCollector();
|
||||
}
|
||||
|
||||
void Context::initializeTraceCollector()
|
||||
{
|
||||
shared->initializeTraceCollector(getTraceLog());
|
||||
|
@ -1077,6 +1077,8 @@ public:
|
||||
void initializeSystemLogs();
|
||||
|
||||
/// Call after initialization before using trace collector.
|
||||
void createTraceCollector();
|
||||
|
||||
void initializeTraceCollector();
|
||||
|
||||
/// Call after unexpected crash happen.
|
||||
|
@ -1,5 +1,4 @@
|
||||
#include "TraceCollector.h"
|
||||
|
||||
#include <Interpreters/TraceCollector.h>
|
||||
#include <Core/Field.h>
|
||||
#include <IO/ReadBufferFromFileDescriptor.h>
|
||||
#include <IO/ReadHelpers.h>
|
||||
@ -14,8 +13,12 @@
|
||||
namespace DB
|
||||
{
|
||||
|
||||
TraceCollector::TraceCollector(std::shared_ptr<TraceLog> trace_log_)
|
||||
: trace_log(std::move(trace_log_))
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int LOGICAL_ERROR;
|
||||
}
|
||||
|
||||
TraceCollector::TraceCollector()
|
||||
{
|
||||
TraceSender::pipe.open();
|
||||
|
||||
@ -28,6 +31,23 @@ TraceCollector::TraceCollector(std::shared_ptr<TraceLog> trace_log_)
|
||||
thread = ThreadFromGlobalPool(&TraceCollector::run, this);
|
||||
}
|
||||
|
||||
void TraceCollector::initialize(std::shared_ptr<TraceLog> trace_log_)
|
||||
{
|
||||
if (is_trace_log_initialized)
|
||||
throw DB::Exception(ErrorCodes::LOGICAL_ERROR, "TraceCollector is already initialized");
|
||||
|
||||
trace_log_ptr = trace_log_;
|
||||
is_trace_log_initialized.store(true, std::memory_order_release);
|
||||
}
|
||||
|
||||
std::shared_ptr<TraceLog> TraceCollector::getTraceLog()
|
||||
{
|
||||
if (!is_trace_log_initialized.load(std::memory_order_acquire))
|
||||
return nullptr;
|
||||
|
||||
return trace_log_ptr;
|
||||
}
|
||||
|
||||
void TraceCollector::tryClosePipe()
|
||||
{
|
||||
try
|
||||
@ -120,7 +140,7 @@ void TraceCollector::run()
|
||||
ProfileEvents::Count increment;
|
||||
readPODBinary(increment, in);
|
||||
|
||||
if (trace_log)
|
||||
if (auto trace_log = getTraceLog())
|
||||
{
|
||||
// time and time_in_microseconds are both being constructed from the same timespec so that the
|
||||
// times will be equal up to the precision of a second.
|
||||
|
@ -1,4 +1,5 @@
|
||||
#pragma once
|
||||
#include <atomic>
|
||||
#include <Common/ThreadPool.h>
|
||||
|
||||
class StackTrace;
|
||||
@ -16,11 +17,17 @@ class TraceLog;
|
||||
class TraceCollector
|
||||
{
|
||||
public:
|
||||
explicit TraceCollector(std::shared_ptr<TraceLog> trace_log_);
|
||||
TraceCollector();
|
||||
~TraceCollector();
|
||||
|
||||
void initialize(std::shared_ptr<TraceLog> trace_log_);
|
||||
|
||||
private:
|
||||
std::shared_ptr<TraceLog> trace_log;
|
||||
std::shared_ptr<TraceLog> getTraceLog();
|
||||
|
||||
std::atomic<bool> is_trace_log_initialized = false;
|
||||
std::shared_ptr<TraceLog> trace_log_ptr;
|
||||
|
||||
ThreadFromGlobalPool thread;
|
||||
|
||||
void tryClosePipe();
|
||||
|
@ -195,12 +195,14 @@ Chunk StorageObjectStorageSource::generate()
|
||||
const auto & object_info = reader.getObjectInfo();
|
||||
const auto & filename = object_info.getFileName();
|
||||
chassert(object_info.metadata);
|
||||
VirtualColumnUtils::addRequestedPathFileAndSizeVirtualsToChunk(
|
||||
chunk,
|
||||
read_from_format_info.requested_virtual_columns,
|
||||
getUniqueStoragePathIdentifier(*configuration, reader.getObjectInfo(), false),
|
||||
object_info.metadata->size_bytes, &filename);
|
||||
|
||||
VirtualColumnUtils::addRequestedFileLikeStorageVirtualsToChunk(
|
||||
chunk, read_from_format_info.requested_virtual_columns,
|
||||
{
|
||||
.path = getUniqueStoragePathIdentifier(*configuration, reader.getObjectInfo(), false),
|
||||
.size = object_info.metadata->size_bytes,
|
||||
.filename = &filename,
|
||||
.last_modified = object_info.metadata->last_modified
|
||||
});
|
||||
return chunk;
|
||||
}
|
||||
|
||||
|
@ -421,8 +421,14 @@ Chunk StorageS3QueueSource::generate()
|
||||
file_status->processed_rows += chunk.getNumRows();
|
||||
processed_rows_from_file += chunk.getNumRows();
|
||||
|
||||
VirtualColumnUtils::addRequestedPathFileAndSizeVirtualsToChunk(
|
||||
chunk, requested_virtual_columns, path, reader.getObjectInfo().metadata->size_bytes);
|
||||
VirtualColumnUtils::addRequestedFileLikeStorageVirtualsToChunk(
|
||||
chunk, requested_virtual_columns,
|
||||
{
|
||||
.path = path,
|
||||
.size = reader.getObjectInfo().metadata->size_bytes
|
||||
});
|
||||
|
||||
|
||||
return chunk;
|
||||
}
|
||||
}
|
||||
|
@ -1341,6 +1341,7 @@ Chunk StorageFileSource::generate()
|
||||
chassert(file_enumerator);
|
||||
current_path = fmt::format("{}::{}", archive_reader->getPath(), *filename_override);
|
||||
current_file_size = file_enumerator->getFileInfo().uncompressed_size;
|
||||
current_file_last_modified = file_enumerator->getFileInfo().last_modified;
|
||||
if (need_only_count && tryGetCountFromCache(current_archive_stat))
|
||||
continue;
|
||||
|
||||
@ -1370,6 +1371,7 @@ Chunk StorageFileSource::generate()
|
||||
struct stat file_stat;
|
||||
file_stat = getFileStat(current_path, storage->use_table_fd, storage->table_fd, storage->getName());
|
||||
current_file_size = file_stat.st_size;
|
||||
current_file_last_modified = Poco::Timestamp::fromEpochTime(file_stat.st_mtime);
|
||||
|
||||
if (getContext()->getSettingsRef().engine_file_skip_empty_files && file_stat.st_size == 0)
|
||||
continue;
|
||||
@ -1436,8 +1438,15 @@ Chunk StorageFileSource::generate()
|
||||
progress(num_rows, chunk_size ? chunk_size : chunk.bytes());
|
||||
|
||||
/// Enrich with virtual columns.
|
||||
VirtualColumnUtils::addRequestedPathFileAndSizeVirtualsToChunk(
|
||||
chunk, requested_virtual_columns, current_path, current_file_size, filename_override.has_value() ? &filename_override.value() : nullptr);
|
||||
VirtualColumnUtils::addRequestedFileLikeStorageVirtualsToChunk(
|
||||
chunk, requested_virtual_columns,
|
||||
{
|
||||
.path = current_path,
|
||||
.size = current_file_size,
|
||||
.filename = (filename_override.has_value() ? &filename_override.value() : nullptr),
|
||||
.last_modified = current_file_last_modified
|
||||
});
|
||||
|
||||
return chunk;
|
||||
}
|
||||
|
||||
|
@ -279,6 +279,7 @@ private:
|
||||
FilesIteratorPtr files_iterator;
|
||||
String current_path;
|
||||
std::optional<size_t> current_file_size;
|
||||
std::optional<Poco::Timestamp> current_file_last_modified;
|
||||
struct stat current_archive_stat;
|
||||
std::optional<String> filename_override;
|
||||
Block sample_block;
|
||||
|
@ -50,6 +50,12 @@ namespace ErrorCodes
|
||||
namespace
|
||||
{
|
||||
|
||||
struct GenerateRandomState
|
||||
{
|
||||
std::atomic<UInt64> add_total_rows = 0;
|
||||
};
|
||||
using GenerateRandomStatePtr = std::shared_ptr<GenerateRandomState>;
|
||||
|
||||
void fillBufferWithRandomData(char * __restrict data, size_t limit, size_t size_of_type, pcg64 & rng, [[maybe_unused]] bool flip_bytes = false)
|
||||
{
|
||||
size_t size = limit * size_of_type;
|
||||
@ -532,10 +538,24 @@ ColumnPtr fillColumnWithRandomData(
|
||||
class GenerateSource : public ISource
|
||||
{
|
||||
public:
|
||||
GenerateSource(UInt64 block_size_, UInt64 max_array_length_, UInt64 max_string_length_, UInt64 random_seed_, Block block_header_, ContextPtr context_)
|
||||
GenerateSource(
|
||||
UInt64 block_size_,
|
||||
UInt64 max_array_length_,
|
||||
UInt64 max_string_length_,
|
||||
UInt64 random_seed_,
|
||||
Block block_header_,
|
||||
ContextPtr context_,
|
||||
GenerateRandomStatePtr state_)
|
||||
: ISource(Nested::flattenNested(prepareBlockToFill(block_header_)))
|
||||
, block_size(block_size_), max_array_length(max_array_length_), max_string_length(max_string_length_)
|
||||
, block_to_fill(std::move(block_header_)), rng(random_seed_), context(context_) {}
|
||||
, block_size(block_size_)
|
||||
, max_array_length(max_array_length_)
|
||||
, max_string_length(max_string_length_)
|
||||
, block_to_fill(std::move(block_header_))
|
||||
, rng(random_seed_)
|
||||
, context(context_)
|
||||
, shared_state(state_)
|
||||
{
|
||||
}
|
||||
|
||||
String getName() const override { return "GenerateRandom"; }
|
||||
|
||||
@ -549,7 +569,15 @@ protected:
|
||||
columns.emplace_back(fillColumnWithRandomData(elem.type, block_size, max_array_length, max_string_length, rng, context));
|
||||
|
||||
columns = Nested::flattenNested(block_to_fill.cloneWithColumns(columns)).getColumns();
|
||||
return {std::move(columns), block_size};
|
||||
|
||||
UInt64 total_rows = shared_state->add_total_rows.fetch_and(0);
|
||||
if (total_rows)
|
||||
addTotalRowsApprox(total_rows);
|
||||
|
||||
auto chunk = Chunk{std::move(columns), block_size};
|
||||
progress(chunk.getNumRows(), chunk.bytes());
|
||||
|
||||
return chunk;
|
||||
}
|
||||
|
||||
private:
|
||||
@ -561,6 +589,7 @@ private:
|
||||
pcg64 rng;
|
||||
|
||||
ContextPtr context;
|
||||
GenerateRandomStatePtr shared_state;
|
||||
|
||||
static Block & prepareBlockToFill(Block & block)
|
||||
{
|
||||
@ -648,9 +677,6 @@ Pipe StorageGenerateRandom::read(
|
||||
{
|
||||
storage_snapshot->check(column_names);
|
||||
|
||||
Pipes pipes;
|
||||
pipes.reserve(num_streams);
|
||||
|
||||
const ColumnsDescription & our_columns = storage_snapshot->metadata->getColumns();
|
||||
Block block_header;
|
||||
for (const auto & name : column_names)
|
||||
@ -679,16 +705,24 @@ Pipe StorageGenerateRandom::read(
|
||||
}
|
||||
}
|
||||
|
||||
UInt64 query_limit = query_info.limit;
|
||||
if (query_limit && num_streams * max_block_size > query_limit)
|
||||
{
|
||||
/// We want to avoid spawning more streams than necessary
|
||||
num_streams = std::min(num_streams, static_cast<size_t>(((query_limit + max_block_size - 1) / max_block_size)));
|
||||
}
|
||||
Pipes pipes;
|
||||
pipes.reserve(num_streams);
|
||||
|
||||
/// Will create more seed values for each source from initial seed.
|
||||
pcg64 generate(random_seed);
|
||||
|
||||
auto shared_state = std::make_shared<GenerateRandomState>(query_info.limit);
|
||||
|
||||
for (UInt64 i = 0; i < num_streams; ++i)
|
||||
{
|
||||
auto source = std::make_shared<GenerateSource>(max_block_size, max_array_length, max_string_length, generate(), block_header, context);
|
||||
|
||||
if (i == 0 && query_info.limit)
|
||||
source->addTotalRowsApprox(query_info.limit);
|
||||
|
||||
auto source = std::make_shared<GenerateSource>(
|
||||
max_block_size, max_array_length, max_string_length, generate(), block_header, context, shared_state);
|
||||
pipes.emplace_back(std::move(source));
|
||||
}
|
||||
|
||||
|
@ -411,7 +411,12 @@ Chunk StorageURLSource::generate()
|
||||
if (input_format)
|
||||
chunk_size = input_format->getApproxBytesReadForChunk();
|
||||
progress(num_rows, chunk_size ? chunk_size : chunk.bytes());
|
||||
VirtualColumnUtils::addRequestedPathFileAndSizeVirtualsToChunk(chunk, requested_virtual_columns, curr_uri.getPath(), current_file_size);
|
||||
VirtualColumnUtils::addRequestedFileLikeStorageVirtualsToChunk(
|
||||
chunk, requested_virtual_columns,
|
||||
{
|
||||
.path = curr_uri.getPath(),
|
||||
.size = current_file_size
|
||||
});
|
||||
return chunk;
|
||||
}
|
||||
|
||||
|
@ -16,7 +16,9 @@ namespace
|
||||
|
||||
struct ZerosState
|
||||
{
|
||||
explicit ZerosState(UInt64 limit) : add_total_rows(limit) { }
|
||||
std::atomic<UInt64> num_generated_rows = 0;
|
||||
std::atomic<UInt64> add_total_rows = 0;
|
||||
};
|
||||
|
||||
using ZerosStatePtr = std::shared_ptr<ZerosState>;
|
||||
@ -42,10 +44,13 @@ protected:
|
||||
auto column_ptr = column;
|
||||
size_t column_size = column_ptr->size();
|
||||
|
||||
if (state)
|
||||
UInt64 total_rows = state->add_total_rows.fetch_and(0);
|
||||
if (total_rows)
|
||||
addTotalRowsApprox(total_rows);
|
||||
|
||||
if (limit)
|
||||
{
|
||||
auto generated_rows = state->num_generated_rows.fetch_add(column_size, std::memory_order_acquire);
|
||||
|
||||
if (generated_rows >= limit)
|
||||
return {};
|
||||
|
||||
@ -103,36 +108,25 @@ Pipe StorageSystemZeros::read(
|
||||
{
|
||||
storage_snapshot->check(column_names);
|
||||
|
||||
bool use_multiple_streams = multithreaded;
|
||||
UInt64 query_limit = limit ? *limit : 0;
|
||||
if (query_info.limit)
|
||||
query_limit = query_limit ? std::min(query_limit, query_info.limit) : query_info.limit;
|
||||
|
||||
if (limit && *limit < max_block_size)
|
||||
{
|
||||
max_block_size = static_cast<size_t>(*limit);
|
||||
use_multiple_streams = false;
|
||||
}
|
||||
if (query_limit && query_limit < max_block_size)
|
||||
max_block_size = query_limit;
|
||||
|
||||
if (!use_multiple_streams)
|
||||
if (!multithreaded)
|
||||
num_streams = 1;
|
||||
else if (query_limit && num_streams * max_block_size > query_limit)
|
||||
/// We want to avoid spawning more streams than necessary
|
||||
num_streams = std::min(num_streams, static_cast<size_t>(((query_limit + max_block_size - 1) / max_block_size)));
|
||||
|
||||
ZerosStatePtr state = std::make_shared<ZerosState>(query_limit);
|
||||
|
||||
Pipe res;
|
||||
|
||||
ZerosStatePtr state;
|
||||
|
||||
if (limit)
|
||||
state = std::make_shared<ZerosState>();
|
||||
|
||||
for (size_t i = 0; i < num_streams; ++i)
|
||||
{
|
||||
auto source = std::make_shared<ZerosSource>(max_block_size, limit ? *limit : 0, state);
|
||||
|
||||
if (i == 0)
|
||||
{
|
||||
if (limit)
|
||||
source->addTotalRowsApprox(*limit);
|
||||
else if (query_info.limit)
|
||||
source->addTotalRowsApprox(query_info.limit);
|
||||
}
|
||||
|
||||
auto source = std::make_shared<ZerosSource>(max_block_size, query_limit, state);
|
||||
res.addSource(std::move(source));
|
||||
}
|
||||
|
||||
|
@ -26,6 +26,7 @@
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <DataTypes/DataTypeLowCardinality.h>
|
||||
#include <DataTypes/DataTypeDateTime.h>
|
||||
|
||||
#include <Processors/QueryPlan/QueryPlan.h>
|
||||
#include <Processors/QueryPlan/BuildQueryPipelineSettings.h>
|
||||
@ -111,7 +112,7 @@ void filterBlockWithDAG(ActionsDAGPtr dag, Block & block, ContextPtr context)
|
||||
|
||||
NameSet getVirtualNamesForFileLikeStorage()
|
||||
{
|
||||
return {"_path", "_file", "_size"};
|
||||
return {"_path", "_file", "_size", "_time"};
|
||||
}
|
||||
|
||||
VirtualColumnsDescription getVirtualsForFileLikeStorage(const ColumnsDescription & storage_columns)
|
||||
@ -129,6 +130,7 @@ VirtualColumnsDescription getVirtualsForFileLikeStorage(const ColumnsDescription
|
||||
add_virtual("_path", std::make_shared<DataTypeLowCardinality>(std::make_shared<DataTypeString>()));
|
||||
add_virtual("_file", std::make_shared<DataTypeLowCardinality>(std::make_shared<DataTypeString>()));
|
||||
add_virtual("_size", makeNullable(std::make_shared<DataTypeUInt64>()));
|
||||
add_virtual("_time", makeNullable(std::make_shared<DataTypeDateTime>()));
|
||||
|
||||
return desc;
|
||||
}
|
||||
@ -187,32 +189,40 @@ ColumnPtr getFilterByPathAndFileIndexes(const std::vector<String> & paths, const
|
||||
return block.getByName("_idx").column;
|
||||
}
|
||||
|
||||
void addRequestedPathFileAndSizeVirtualsToChunk(
|
||||
Chunk & chunk, const NamesAndTypesList & requested_virtual_columns, const String & path, std::optional<size_t> size, const String * filename)
|
||||
void addRequestedFileLikeStorageVirtualsToChunk(
|
||||
Chunk & chunk, const NamesAndTypesList & requested_virtual_columns,
|
||||
VirtualsForFileLikeStorage virtual_values)
|
||||
{
|
||||
for (const auto & virtual_column : requested_virtual_columns)
|
||||
{
|
||||
if (virtual_column.name == "_path")
|
||||
{
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), path)->convertToFullColumnIfConst());
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), virtual_values.path)->convertToFullColumnIfConst());
|
||||
}
|
||||
else if (virtual_column.name == "_file")
|
||||
{
|
||||
if (filename)
|
||||
if (virtual_values.filename)
|
||||
{
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), *filename)->convertToFullColumnIfConst());
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), (*virtual_values.filename))->convertToFullColumnIfConst());
|
||||
}
|
||||
else
|
||||
{
|
||||
size_t last_slash_pos = path.find_last_of('/');
|
||||
auto filename_from_path = path.substr(last_slash_pos + 1);
|
||||
size_t last_slash_pos = virtual_values.path.find_last_of('/');
|
||||
auto filename_from_path = virtual_values.path.substr(last_slash_pos + 1);
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), filename_from_path)->convertToFullColumnIfConst());
|
||||
}
|
||||
}
|
||||
else if (virtual_column.name == "_size")
|
||||
{
|
||||
if (size)
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), *size)->convertToFullColumnIfConst());
|
||||
if (virtual_values.size)
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), *virtual_values.size)->convertToFullColumnIfConst());
|
||||
else
|
||||
chunk.addColumn(virtual_column.type->createColumnConstWithDefaultValue(chunk.getNumRows())->convertToFullColumnIfConst());
|
||||
}
|
||||
else if (virtual_column.name == "_time")
|
||||
{
|
||||
if (virtual_values.last_modified)
|
||||
chunk.addColumn(virtual_column.type->createColumnConst(chunk.getNumRows(), virtual_values.last_modified->epochTime())->convertToFullColumnIfConst());
|
||||
else
|
||||
chunk.addColumn(virtual_column.type->createColumnConstWithDefaultValue(chunk.getNumRows())->convertToFullColumnIfConst());
|
||||
}
|
||||
|
@ -68,8 +68,18 @@ void filterByPathOrFile(std::vector<T> & sources, const std::vector<String> & pa
|
||||
sources = std::move(filtered_sources);
|
||||
}
|
||||
|
||||
void addRequestedPathFileAndSizeVirtualsToChunk(
|
||||
Chunk & chunk, const NamesAndTypesList & requested_virtual_columns, const String & path, std::optional<size_t> size, const String * filename = nullptr);
|
||||
struct VirtualsForFileLikeStorage
|
||||
{
|
||||
const String & path;
|
||||
std::optional<size_t> size { std::nullopt };
|
||||
const String * filename { nullptr };
|
||||
std::optional<Poco::Timestamp> last_modified { std::nullopt };
|
||||
|
||||
};
|
||||
|
||||
void addRequestedFileLikeStorageVirtualsToChunk(
|
||||
Chunk & chunk, const NamesAndTypesList & requested_virtual_columns,
|
||||
VirtualsForFileLikeStorage virtual_values);
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -685,9 +685,6 @@ class CIConfig:
|
||||
return result
|
||||
|
||||
def get_job_parents(self, check_name: str) -> List[str]:
|
||||
if check_name in self.builds_report_config:
|
||||
return self.builds_report_config[check_name].builds
|
||||
|
||||
res = []
|
||||
check_name = normalize_string(check_name)
|
||||
for config in (
|
||||
@ -903,10 +900,38 @@ CI_CONFIG = CIConfig(
|
||||
),
|
||||
CILabels.CI_SET_REQUIRED: LabelConfig(run_jobs=REQUIRED_CHECKS),
|
||||
CILabels.CI_SET_NORMAL_BUILDS: LabelConfig(
|
||||
run_jobs=[JobNames.STYLE_CHECK, JobNames.BUILD_CHECK]
|
||||
run_jobs=[
|
||||
JobNames.STYLE_CHECK,
|
||||
JobNames.BUILD_CHECK,
|
||||
Build.PACKAGE_RELEASE,
|
||||
Build.PACKAGE_AARCH64,
|
||||
Build.PACKAGE_ASAN,
|
||||
Build.PACKAGE_UBSAN,
|
||||
Build.PACKAGE_TSAN,
|
||||
Build.PACKAGE_MSAN,
|
||||
Build.PACKAGE_DEBUG,
|
||||
Build.BINARY_RELEASE,
|
||||
Build.PACKAGE_RELEASE_COVERAGE,
|
||||
Build.FUZZERS,
|
||||
]
|
||||
),
|
||||
CILabels.CI_SET_SPECIAL_BUILDS: LabelConfig(
|
||||
run_jobs=[JobNames.STYLE_CHECK, JobNames.BUILD_CHECK_SPECIAL]
|
||||
run_jobs=[
|
||||
JobNames.STYLE_CHECK,
|
||||
JobNames.BUILD_CHECK_SPECIAL,
|
||||
Build.BINARY_TIDY,
|
||||
Build.BINARY_DARWIN,
|
||||
Build.BINARY_AARCH64,
|
||||
Build.BINARY_AARCH64_V80COMPAT,
|
||||
Build.BINARY_FREEBSD,
|
||||
Build.BINARY_DARWIN_AARCH64,
|
||||
Build.BINARY_PPC64LE,
|
||||
Build.BINARY_RISCV64,
|
||||
Build.BINARY_S390X,
|
||||
Build.BINARY_LOONGARCH64,
|
||||
Build.BINARY_AMD64_COMPAT,
|
||||
Build.BINARY_AMD64_MUSL,
|
||||
]
|
||||
),
|
||||
CILabels.CI_SET_NON_REQUIRED: LabelConfig(
|
||||
run_jobs=[job for job in JobNames if job not in REQUIRED_CHECKS]
|
||||
|
@ -758,12 +758,12 @@ def test_read_subcolumns(cluster):
|
||||
)
|
||||
|
||||
res = node.query(
|
||||
f"select a.b.d, _path, a.b, _file, a.e from azureBlobStorage('{storage_account_url}', 'cont', 'test_subcolumns.tsv',"
|
||||
f"select a.b.d, _path, a.b, _file, dateDiff('minute', _time, now()), a.e from azureBlobStorage('{storage_account_url}', 'cont', 'test_subcolumns.tsv',"
|
||||
f" 'devstoreaccount1', 'Eby8vdM02xNOcqFlqUwJPLlmEtlCDXJ1OUzFT50uSRZ6IFsuFq2UVErCz4I6tq/K1SZFPTOtr/KBHBeksoGMGw==', 'auto', 'auto',"
|
||||
f" 'a Tuple(b Tuple(c UInt32, d UInt32), e UInt32)')"
|
||||
)
|
||||
|
||||
assert res == "2\tcont/test_subcolumns.tsv\t(1,2)\ttest_subcolumns.tsv\t3\n"
|
||||
assert res == "2\tcont/test_subcolumns.tsv\t(1,2)\ttest_subcolumns.tsv\t0\t3\n"
|
||||
|
||||
res = node.query(
|
||||
f"select a.b.d, _path, a.b, _file, a.e from azureBlobStorage('{storage_account_url}', 'cont', 'test_subcolumns.jsonl',"
|
||||
|
@ -987,10 +987,10 @@ def test_read_subcolumns(started_cluster):
|
||||
assert res == "2\ttest_subcolumns.jsonl\t(1,2)\ttest_subcolumns.jsonl\t3\n"
|
||||
|
||||
res = node.query(
|
||||
f"select x.b.d, _path, x.b, _file, x.e from hdfs('hdfs://hdfs1:9000/test_subcolumns.jsonl', auto, 'x Tuple(b Tuple(c UInt32, d UInt32), e UInt32)')"
|
||||
f"select x.b.d, _path, x.b, _file, dateDiff('minute', _time, now()), x.e from hdfs('hdfs://hdfs1:9000/test_subcolumns.jsonl', auto, 'x Tuple(b Tuple(c UInt32, d UInt32), e UInt32)')"
|
||||
)
|
||||
|
||||
assert res == "0\ttest_subcolumns.jsonl\t(0,0)\ttest_subcolumns.jsonl\t0\n"
|
||||
assert res == "0\ttest_subcolumns.jsonl\t(0,0)\ttest_subcolumns.jsonl\t0\t0\n"
|
||||
|
||||
res = node.query(
|
||||
f"select x.b.d, _path, x.b, _file, x.e from hdfs('hdfs://hdfs1:9000/test_subcolumns.jsonl', auto, 'x Tuple(b Tuple(c UInt32, d UInt32), e UInt32) default ((42, 42), 42)')"
|
||||
|
@ -2117,10 +2117,12 @@ def test_read_subcolumns(started_cluster):
|
||||
assert res == "0\troot/test_subcolumns.jsonl\t(0,0)\ttest_subcolumns.jsonl\t0\n"
|
||||
|
||||
res = instance.query(
|
||||
f"select x.b.d, _path, x.b, _file, x.e from s3('http://{started_cluster.minio_host}:{started_cluster.minio_port}/{bucket}/test_subcolumns.jsonl', auto, 'x Tuple(b Tuple(c UInt32, d UInt32), e UInt32) default ((42, 42), 42)')"
|
||||
f"select x.b.d, _path, x.b, _file, dateDiff('minute', _time, now()), x.e from s3('http://{started_cluster.minio_host}:{started_cluster.minio_port}/{bucket}/test_subcolumns.jsonl', auto, 'x Tuple(b Tuple(c UInt32, d UInt32), e UInt32) default ((42, 42), 42)')"
|
||||
)
|
||||
|
||||
assert res == "42\troot/test_subcolumns.jsonl\t(42,42)\ttest_subcolumns.jsonl\t42\n"
|
||||
assert (
|
||||
res == "42\troot/test_subcolumns.jsonl\t(42,42)\ttest_subcolumns.jsonl\t0\t42\n"
|
||||
)
|
||||
|
||||
res = instance.query(
|
||||
f"select a.b.d, _path, a.b, _file, a.e from url('http://{started_cluster.minio_host}:{started_cluster.minio_port}/{bucket}/test_subcolumns.tsv', auto, 'a Tuple(b Tuple(c UInt32, d UInt32), e UInt32)')"
|
||||
@ -2148,6 +2150,8 @@ def test_read_subcolumns(started_cluster):
|
||||
res == "42\t/root/test_subcolumns.jsonl\t(42,42)\ttest_subcolumns.jsonl\t42\n"
|
||||
)
|
||||
|
||||
logging.info("Some custom logging")
|
||||
|
||||
|
||||
def test_filtering_by_file_or_path(started_cluster):
|
||||
bucket = started_cluster.minio_bucket
|
||||
|
@ -1,5 +1,5 @@
|
||||
#!/usr/bin/env bash
|
||||
# Tags: long, zookeeper, no-parallel, no-fasttest
|
||||
# Tags: long, zookeeper, no-parallel, no-fasttest, no-asan
|
||||
|
||||
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
|
@ -1 +0,0 @@
|
||||
1 2 4
|
@ -1,13 +0,0 @@
|
||||
#!/usr/bin/env bash
|
||||
|
||||
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
. "$CURDIR"/../shell_config.sh
|
||||
|
||||
echo "1,2" > $CLICKHOUSE_TEST_UNIQUE_NAME.csv
|
||||
$CLICKHOUSE_LOCAL -nm -q "
|
||||
create table test (x UInt64, y UInt32, size UInt64) engine=Memory;
|
||||
insert into test select c1, c2, _size from file('$CLICKHOUSE_TEST_UNIQUE_NAME.csv') settings use_structure_from_insertion_table_in_table_functions=1;
|
||||
select * from test;
|
||||
"
|
||||
rm $CLICKHOUSE_TEST_UNIQUE_NAME.csv
|
@ -0,0 +1 @@
|
||||
1 2 4 1 1
|
@ -0,0 +1,14 @@
|
||||
#!/usr/bin/env bash
|
||||
|
||||
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
. "$CURDIR"/../shell_config.sh
|
||||
|
||||
echo "1,2" > $CLICKHOUSE_TEST_UNIQUE_NAME.csv
|
||||
sleep 1
|
||||
$CLICKHOUSE_LOCAL -nm -q "
|
||||
create table test (x UInt64, y UInt32, size UInt64, d32 DateTime32, d64 DateTime64) engine=Memory;
|
||||
insert into test select c1, c2, _size, _time, _time from file('$CLICKHOUSE_TEST_UNIQUE_NAME.csv') settings use_structure_from_insertion_table_in_table_functions=1;
|
||||
select x, y, size, (dateDiff('millisecond', d32, now()) < 4000 AND dateDiff('millisecond', d32, now()) > 0), (dateDiff('second', d64, now()) < 4 AND dateDiff('second', d64, now()) > 0) from test;
|
||||
"
|
||||
rm $CLICKHOUSE_TEST_UNIQUE_NAME.csv
|
@ -1,49 +0,0 @@
|
||||
#!/usr/bin/expect -f
|
||||
|
||||
set basedir [file dirname $argv0]
|
||||
set basename [file tail $argv0]
|
||||
if {[info exists env(CLICKHOUSE_TMP)]} {
|
||||
set CLICKHOUSE_TMP $env(CLICKHOUSE_TMP)
|
||||
} else {
|
||||
set CLICKHOUSE_TMP "."
|
||||
}
|
||||
exp_internal -f $CLICKHOUSE_TMP/$basename.debuglog 0
|
||||
|
||||
log_user 0
|
||||
set timeout 60
|
||||
match_max 100000
|
||||
set stty_init "rows 25 cols 120"
|
||||
|
||||
expect_after {
|
||||
-i $any_spawn_id eof { exp_continue }
|
||||
-i $any_spawn_id timeout { exit 1 }
|
||||
}
|
||||
|
||||
spawn clickhouse-local
|
||||
expect ":) "
|
||||
|
||||
# Trivial SELECT with LIMIT from system.zeros shows progress bar.
|
||||
send "SELECT * FROM system.zeros LIMIT 10000000 FORMAT Null SETTINGS max_execution_speed = 1000000, timeout_before_checking_execution_speed = 0, max_block_size = 128\r"
|
||||
expect "Progress: "
|
||||
expect "█"
|
||||
send "\3"
|
||||
expect "Query was cancelled."
|
||||
expect ":) "
|
||||
|
||||
send "SELECT * FROM system.zeros_mt LIMIT 10000000 FORMAT Null SETTINGS max_execution_speed = 1000000, timeout_before_checking_execution_speed = 0, max_block_size = 128\r"
|
||||
expect "Progress: "
|
||||
expect "█"
|
||||
send "\3"
|
||||
expect "Query was cancelled."
|
||||
expect ":) "
|
||||
|
||||
# As well as from generateRandom
|
||||
send "SELECT * FROM generateRandom() LIMIT 10000000 FORMAT Null SETTINGS max_execution_speed = 1000000, timeout_before_checking_execution_speed = 0, max_block_size = 128\r"
|
||||
expect "Progress: "
|
||||
expect "█"
|
||||
send "\3"
|
||||
expect "Query was cancelled."
|
||||
expect ":) "
|
||||
|
||||
send "exit\r"
|
||||
expect eof
|
@ -0,0 +1,3 @@
|
||||
Matched
|
||||
Matched
|
||||
Matched
|
@ -0,0 +1,18 @@
|
||||
#!/usr/bin/env bash
|
||||
# Tags: no-random-settings
|
||||
|
||||
CUR_DIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
. "$CUR_DIR"/../shell_config.sh
|
||||
|
||||
function run_with_progress_and_match_total_rows()
|
||||
{
|
||||
CURL_RESPONSE=$(echo "$1" | \
|
||||
${CLICKHOUSE_CURL} -vsS "${CLICKHOUSE_URL}&wait_end_of_query=1&max_block_size=1&send_progress_in_http_headers=1&http_headers_progress_interval_ms=0&output_format_parallel_formatting=0" --data-binary @- 2>&1)
|
||||
|
||||
echo "$CURL_RESPONSE" | grep -q '"total_rows_to_read":"100"' && echo "Matched" || echo "Expected total_rows_to_read not found: ${CURL_RESPONSE}"
|
||||
}
|
||||
|
||||
run_with_progress_and_match_total_rows 'SELECT * FROM system.zeros LIMIT 100'
|
||||
run_with_progress_and_match_total_rows 'SELECT * FROM system.zeros_mt LIMIT 100'
|
||||
run_with_progress_and_match_total_rows "SELECT * FROM generateRandom('number UInt64') LIMIT 100"
|
2
tests/queries/0_stateless/03168_cld2_tsan.reference
Normal file
2
tests/queries/0_stateless/03168_cld2_tsan.reference
Normal file
@ -0,0 +1,2 @@
|
||||
{'ja':0.62,'fr':0.36}
|
||||
{'ja':0.62,'fr':0.36}
|
10
tests/queries/0_stateless/03168_cld2_tsan.sql
Normal file
10
tests/queries/0_stateless/03168_cld2_tsan.sql
Normal file
@ -0,0 +1,10 @@
|
||||
-- Tags: no-fasttest
|
||||
-- Tag no-fasttest: depends on cld2
|
||||
|
||||
-- https://github.com/ClickHouse/ClickHouse/issues/64931
|
||||
SELECT detectLanguageMixed(materialize('二兎を追う者は一兎をも得ず二兎を追う者は一兎をも得ず A vaincre sans peril, on triomphe sans gloire.'))
|
||||
GROUP BY
|
||||
GROUPING SETS (
|
||||
('a', toUInt256(1)),
|
||||
(stringToH3(toFixedString(toFixedString('85283473ffffff', 14), 14))))
|
||||
SETTINGS allow_experimental_nlp_functions = 1;
|
@ -0,0 +1,6 @@
|
||||
-- https://github.com/ClickHouse/ClickHouse/issues/64946
|
||||
SELECT
|
||||
multiIf((number % toLowCardinality(toNullable(toUInt128(2)))) = (number % toNullable(2)), toInt8(1), (number % materialize(toLowCardinality(3))) = toUInt128(toNullable(0)), toInt8(materialize(materialize(2))), toInt64(toUInt128(3)))
|
||||
FROM system.numbers
|
||||
LIMIT 44857
|
||||
FORMAT Null;
|
Loading…
Reference in New Issue
Block a user