Merge branch 'master' into default_on_aliases

This commit is contained in:
Alexey Milovidov 2020-03-05 20:42:30 +03:00
commit d4dcd1a2d5
245 changed files with 4411 additions and 1856 deletions

View File

@ -52,7 +52,7 @@ IncludeCategories:
ReflowComments: false
AlignEscapedNewlinesLeft: false
AlignEscapedNewlines: DontAlign
AlignTrailingComments: true
AlignTrailingComments: false
# Not changed:
AccessModifierOffset: -4

View File

@ -80,7 +80,6 @@ dumpImpl(Out & out, T && x)
}
/// Tuple, pair
template <size_t N, typename Out, typename T>
Out & dumpTupleImpl(Out & out, T && x)

View File

@ -1,44 +0,0 @@
#pragma once
#include <memory>
namespace ext
{
/** Thread-unsafe singleton. It works simply like a global variable.
* Supports deinitialization.
*
* In most of the cases, you don't need this class.
* Use "Meyers Singleton" instead: static T & instance() { static T x; return x; }
*/
template <class T>
class Singleton
{
public:
Singleton()
{
if (!instance)
instance = std::make_unique<T>();
}
T * operator->()
{
return instance.get();
}
static bool isInitialized()
{
return !!instance;
}
static void reset()
{
instance.reset();
}
private:
inline static std::unique_ptr<T> instance{};
};
}

View File

@ -2,7 +2,6 @@
#include <stdint.h>
#include <time.h>
#include "atomic.h"
#include "musl_features.h"
#include "syscall.h"
#ifdef VDSO_CGT_SYM
@ -54,7 +53,7 @@ static void *volatile vdso_func = (void *)cgt_init;
#endif
int __clock_gettime(clockid_t clk, struct timespec *ts)
int clock_gettime(clockid_t clk, struct timespec *ts)
{
int r;
@ -104,5 +103,3 @@ int __clock_gettime(clockid_t clk, struct timespec *ts)
return __syscall_ret(r);
#endif
}
weak_alias(__clock_gettime, clock_gettime);

View File

@ -1,10 +1,9 @@
#include <errno.h>
#include <pthread.h>
#include <time.h>
#include "musl_features.h"
#include "syscall.h"
int __clock_nanosleep(clockid_t clk, int flags, const struct timespec * req, struct timespec * rem)
int clock_nanosleep(clockid_t clk, int flags, const struct timespec * req, struct timespec * rem)
{
if (clk == CLOCK_THREAD_CPUTIME_ID)
return EINVAL;
@ -23,5 +22,3 @@ int __clock_nanosleep(clockid_t clk, int flags, const struct timespec * req, str
pthread_setcanceltype(old_cancel_type, NULL);
return status;
}
weak_alias(__clock_nanosleep, clock_nanosleep);

View File

@ -2,7 +2,4 @@
#define weak __attribute__((__weak__))
#define hidden __attribute__((__visibility__("hidden")))
#define weak_alias(old, new) \
extern __typeof(old) new __attribute__((__weak__, __alias__(#old)))
#define predict_false(x) __builtin_expect(x, 0)

View File

@ -2,6 +2,7 @@
.hidden __syscall
.type __syscall,@function
__syscall:
.cfi_startproc
movq %rdi,%rax
movq %rsi,%rdi
movq %rdx,%rsi
@ -11,3 +12,4 @@ __syscall:
movq 8(%rsp),%r9
syscall
ret
.cfi_endproc

View File

@ -39,7 +39,6 @@ typedef __attribute__((__aligned__(1))) uint32_t uint32_unaligned_t;
typedef __attribute__((__aligned__(1))) uint64_t uint64_unaligned_t;
//---------------------------------------------------------------------
// fast copy for different sizes
//---------------------------------------------------------------------
@ -694,4 +693,3 @@ static INLINE void* memcpy_fast(void *destination, const void *source, size_t si
#endif

2
contrib/base64 vendored

@ -1 +1 @@
Subproject commit 5257626d2be17a3eb23f79be17fe55ebba394ad2
Subproject commit 95ba56a9b041f9933f5cd2bbb2ee4e083468c20a

2
contrib/poco vendored

@ -1 +1 @@
Subproject commit 1f3e4638f250ad4d028a2499af20d4185463e07d
Subproject commit 860574c93980d887a89df141edd9ca2fb0024fa3

View File

@ -131,7 +131,6 @@ struct TaskStateWithOwner
};
struct ShardPriority
{
UInt8 is_remote = 1;

View File

@ -298,7 +298,7 @@ void LocalServer::processQueries()
try
{
executeQuery(read_buf, write_buf, /* allow_into_outfile = */ true, *context, {}, {});
executeQuery(read_buf, write_buf, /* allow_into_outfile = */ true, *context, {});
}
catch (...)
{

View File

@ -60,5 +60,4 @@ void StopConditionsSet::report(UInt64 value, StopConditionsSet::StopCondition &
}
}

View File

@ -592,12 +592,14 @@ void HTTPHandler::processQuery(
customizeContext(context);
executeQuery(*in, *used_output.out_maybe_delayed_and_compressed, /* allow_into_outfile = */ false, context,
[&response] (const String & content_type, const String & format)
[&response] (const String & current_query_id, const String & content_type, const String & format, const String & timezone)
{
response.setContentType(content_type);
response.add("X-ClickHouse-Query-Id", current_query_id);
response.add("X-ClickHouse-Format", format);
},
[&response] (const String & current_query_id) { response.add("X-ClickHouse-Query-Id", current_query_id); });
response.add("X-ClickHouse-Timezone", timezone);
}
);
if (used_output.hasDelayed())
{

View File

@ -282,14 +282,9 @@ void MySQLHandler::comQuery(ReadBuffer & payload)
}
else
{
bool with_output = false;
std::function<void(const String &, const String &)> set_content_type_and_format = [&with_output](const String &, const String &) -> void
{
with_output = true;
};
String replacement_query = "select ''";
bool should_replace = false;
bool with_output = false;
// Translate query from MySQL to ClickHouse.
// This is a temporary workaround until ClickHouse supports the syntax "@@var_name".
@ -307,7 +302,13 @@ void MySQLHandler::comQuery(ReadBuffer & payload)
ReadBufferFromString replacement(replacement_query);
Context query_context = connection_context;
executeQuery(should_replace ? replacement : payload, *out, true, query_context, set_content_type_and_format, {});
executeQuery(should_replace ? replacement : payload, *out, true, query_context,
[&with_output](const String &, const String &, const String &, const String &)
{
with_output = true;
}
);
if (!with_output)
packet_sender->sendPacket(OK_Packet(0x00, client_capability_flags, 0, 0, 0), true);

View File

@ -60,6 +60,7 @@
#include "TCPHandlerFactory.h"
#include "Common/config_version.h"
#include <Common/SensitiveDataMasker.h>
#include <Common/ThreadFuzzer.h>
#include "MySQLHandlerFactory.h"
#if defined(OS_LINUX)
@ -219,6 +220,9 @@ int Server::main(const std::vector<std::string> & /*args*/)
CurrentMetrics::set(CurrentMetrics::Revision, ClickHouseRevision::get());
CurrentMetrics::set(CurrentMetrics::VersionInteger, ClickHouseRevision::getVersionInteger());
if (ThreadFuzzer::instance().isEffective())
LOG_WARNING(log, "ThreadFuzzer is enabled. Application will run slowly and unstable.");
/** Context contains all that query execution is dependent:
* settings, available functions, data types, aggregate functions, databases...
*/
@ -466,6 +470,8 @@ int Server::main(const std::vector<std::string> & /*args*/)
if (config->has("max_partition_size_to_drop"))
global_context->setMaxPartitionSizeToDrop(config->getUInt64("max_partition_size_to_drop"));
global_context->updateStorageConfiguration(*config);
},
/* already_loaded = */ true);

View File

@ -503,7 +503,7 @@ void TCPHandler::processOrdinaryQuery()
if (after_send_progress.elapsed() / 1000 >= query_context->getSettingsRef().interactive_delay)
{
/// Some time passed and there is a progress.
/// Some time passed.
after_send_progress.restart();
sendProgress();
}
@ -539,6 +539,8 @@ void TCPHandler::processOrdinaryQuery()
}
state.io.onFinish();
sendProgress();
}
@ -546,8 +548,8 @@ void TCPHandler::processOrdinaryQueryWithProcessors(size_t num_threads)
{
auto & pipeline = state.io.pipeline;
if (pipeline.getMaxThreads())
num_threads = std::min(num_threads, pipeline.getMaxThreads());
/// Reduce the number of threads to recommended value.
num_threads = std::min(num_threads, pipeline.getNumThreads());
/// Send header-block, to allow client to prepare output format for data to send.
{
@ -658,6 +660,8 @@ void TCPHandler::processOrdinaryQueryWithProcessors(size_t num_threads)
}
state.io.onFinish();
sendProgress();
}

View File

@ -72,7 +72,6 @@ bool IAccessStorage::exists(const UUID & id) const
}
AccessEntityPtr IAccessStorage::tryReadBase(const UUID & id) const
{
try

View File

@ -5,7 +5,6 @@
#include <chrono>
namespace DB
{
/** Quota for resources consumption for specific interval.

View File

@ -86,7 +86,6 @@ struct MovingAvgData
};
template <typename T, typename Tlimit_num_elems, typename Data>
class MovingImpl final
: public IAggregateFunctionDataHelper<Data, MovingImpl<T, Tlimit_num_elems, Data>>

View File

@ -158,6 +158,21 @@ void ColumnAggregateFunction::ensureOwnership()
}
bool ColumnAggregateFunction::structureEquals(const IColumn & to) const
{
const auto * to_concrete = typeid_cast<const ColumnAggregateFunction *>(&to);
if (!to_concrete)
return false;
/// AggregateFunctions must be the same.
const IAggregateFunction & func_this = *func;
const IAggregateFunction & func_to = *to_concrete->func;
return typeid(func_this) == typeid(func_to);
}
void ColumnAggregateFunction::insertRangeFrom(const IColumn & from, size_t start, size_t length)
{
const ColumnAggregateFunction & from_concrete = assert_cast<const ColumnAggregateFunction &>(from);

View File

@ -204,6 +204,8 @@ public:
}
void getExtremes(Field & min, Field & max) const override;
bool structureEquals(const IColumn &) const override;
};

View File

@ -287,5 +287,4 @@ private:
};
}

View File

@ -339,7 +339,7 @@ ColumnPtr ColumnVector<T>::index(const IColumn & indexes, size_t limit) const
template <typename T>
ColumnPtr ColumnVector<T>::replicate(const IColumn::Offsets & offsets) const
{
size_t size = data.size();
const size_t size = data.size();
if (size != offsets.size())
throw Exception("Size of offsets doesn't match size of column.", ErrorCodes::SIZES_OF_COLUMNS_DOESNT_MATCH);
@ -352,7 +352,7 @@ ColumnPtr ColumnVector<T>::replicate(const IColumn::Offsets & offsets) const
for (size_t i = 0; i < size; ++i)
{
const auto span_end = res->getData().begin() + offsets[i];
for (; it < span_end; ++it)
for (; it != span_end; ++it)
*it = data[i];
}

View File

@ -488,6 +488,7 @@ namespace ErrorCodes
extern const int ACCESS_STORAGE_FOR_INSERTION_NOT_FOUND = 514;
extern const int INCORRECT_ACCESS_ENTITY_DEFINITION = 515;
extern const int AUTHENTICATION_FAILED = 516;
extern const int CANNOT_ASSIGN_ALTER = 517;
extern const int KEEPER_EXCEPTION = 999;
extern const int POCO_EXCEPTION = 1000;

View File

@ -6,7 +6,6 @@
#include <map>
namespace Poco
{
namespace Util

View File

@ -7,7 +7,6 @@
#include <Common/formatReadable.h>
#include <common/likely.h>
#include <common/logger_useful.h>
#include <ext/singleton.h>
#include <atomic>
#include <cmath>
@ -71,14 +70,13 @@ static void logMemoryUsage(Int64 amount)
}
void MemoryTracker::alloc(Int64 size)
{
if (blocker.isCancelled())
return;
/** Using memory_order_relaxed means that if allocations are done simultaneously,
* we allow exception about memory limit exceeded to be thrown only on next allocation.
* we allow exception about memory limit exceeded to be thrown only on next allocation.
* So, we allow over-allocations.
*/
Int64 will_be = size + amount.fetch_add(size, std::memory_order_relaxed);
@ -112,8 +110,8 @@ void MemoryTracker::alloc(Int64 size)
if (unlikely(current_profiler_limit && will_be > current_profiler_limit))
{
auto no_track = blocker.cancel();
ext::Singleton<DB::TraceCollector>()->collect(size);
setOrRaiseProfilerLimit(current_profiler_limit + Int64(std::ceil((will_be - current_profiler_limit) / profiler_step)) * profiler_step);
DB::TraceCollector::collect(DB::TraceType::Memory, StackTrace(), size);
setOrRaiseProfilerLimit((will_be + profiler_step - 1) / profiler_step * profiler_step);
}
if (unlikely(current_hard_limit && will_be > current_hard_limit))
@ -212,7 +210,6 @@ void MemoryTracker::setOrRaiseHardLimit(Int64 value)
void MemoryTracker::setOrRaiseProfilerLimit(Int64 value)
{
/// This is just atomic set to maximum.
Int64 old_value = profiler_limit.load(std::memory_order_relaxed);
while (old_value < value && !profiler_limit.compare_exchange_weak(old_value, value))
;

View File

@ -7,7 +7,6 @@
#include <memory>
#include <boost/noncopyable.hpp>
#include <boost/iterator_adaptors.hpp>
#include <common/likely.h>
#include <common/strong_typedef.h>
@ -275,18 +274,11 @@ protected:
public:
using value_type = T;
/// You can not just use `typedef`, because there is ambiguity for the constructors and `assign` functions.
struct iterator : public boost::iterator_adaptor<iterator, T*>
{
iterator() {}
iterator(T * ptr_) : iterator::iterator_adaptor_(ptr_) {}
};
/// We cannot use boost::iterator_adaptor, because it defeats loop vectorization,
/// see https://github.com/ClickHouse/ClickHouse/pull/9442
struct const_iterator : public boost::iterator_adaptor<const_iterator, const T*>
{
const_iterator() {}
const_iterator(const T * ptr_) : const_iterator::iterator_adaptor_(ptr_) {}
};
using iterator = T *;
using const_iterator = const T *;
PODArray() {}

View File

@ -9,24 +9,48 @@
#include <common/config_common.h>
#include <common/logger_useful.h>
#include <common/phdr_cache.h>
#include <ext/singleton.h>
#include <random>
namespace ProfileEvents
{
extern const Event QueryProfilerSignalOverruns;
}
namespace DB
{
namespace
{
thread_local size_t write_trace_iteration = 0;
void writeTraceInfo(TraceType trace_type, int /* sig */, siginfo_t * info, void * context)
{
auto saved_errno = errno; /// We must restore previous value of errno in signal handler.
int overrun_count = 0;
#if defined(OS_LINUX)
if (info)
overrun_count = info->si_overrun;
{
int overrun_count = info->si_overrun;
/// Quickly drop if signal handler is called too frequently.
/// Otherwise we may end up infinitelly processing signals instead of doing any useful work.
++write_trace_iteration;
if (overrun_count)
{
/// But pass with some frequency to avoid drop of all traces.
if (write_trace_iteration % overrun_count == 0)
{
ProfileEvents::increment(ProfileEvents::QueryProfilerSignalOverruns, overrun_count);
}
else
{
ProfileEvents::increment(ProfileEvents::QueryProfilerSignalOverruns, overrun_count + 1);
return;
}
}
}
#else
UNUSED(info);
#endif
@ -34,12 +58,12 @@ namespace
const auto signal_context = *reinterpret_cast<ucontext_t *>(context);
const StackTrace stack_trace(signal_context);
ext::Singleton<TraceCollector>()->collect(trace_type, stack_trace, overrun_count);
TraceCollector::collect(trace_type, stack_trace, 0);
errno = saved_errno;
}
[[maybe_unused]] const UInt32 TIMER_PRECISION = 1e9;
[[maybe_unused]] constexpr UInt32 TIMER_PRECISION = 1e9;
}
namespace ErrorCodes
@ -81,7 +105,7 @@ QueryProfilerBase<ProfilerImpl>::QueryProfilerBase(const UInt64 thread_id, const
try
{
struct sigevent sev;
struct sigevent sev {};
sev.sigev_notify = SIGEV_THREAD_ID;
sev.sigev_signo = pause_signal;
@ -156,7 +180,7 @@ QueryProfilerReal::QueryProfilerReal(const UInt64 thread_id, const UInt32 period
void QueryProfilerReal::signalHandler(int sig, siginfo_t * info, void * context)
{
writeTraceInfo(TraceType::REAL_TIME, sig, info, context);
writeTraceInfo(TraceType::Real, sig, info, context);
}
QueryProfilerCpu::QueryProfilerCpu(const UInt64 thread_id, const UInt32 period)
@ -165,7 +189,7 @@ QueryProfilerCpu::QueryProfilerCpu(const UInt64 thread_id, const UInt32 period)
void QueryProfilerCpu::signalHandler(int sig, siginfo_t * info, void * context)
{
writeTraceInfo(TraceType::CPU_TIME, sig, info, context);
writeTraceInfo(TraceType::CPU, sig, info, context);
}
}

View File

@ -109,7 +109,6 @@ struct RadixSortIdentityTransform
};
template <typename TElement>
struct RadixSortUIntTraits
{

View File

@ -0,0 +1,133 @@
#include <signal.h>
#include <time.h>
#include <sys/time.h>
#if OS_LINUX
#include <sys/sysinfo.h>
#endif
#include <sched.h>
#include <random>
#include <common/sleep.h>
#include <common/getThreadId.h>
#include <IO/ReadHelpers.h>
#include <Common/Exception.h>
#include <Common/thread_local_rng.h>
#include <Common/ThreadFuzzer.h>
namespace DB
{
namespace ErrorCodes
{
extern const int CANNOT_MANIPULATE_SIGSET;
extern const int CANNOT_SET_SIGNAL_HANDLER;
extern const int CANNOT_CREATE_TIMER;
}
ThreadFuzzer::ThreadFuzzer()
{
initConfiguration();
if (!isEffective())
return;
setup();
}
template <typename T>
static void initFromEnv(T & what, const char * name)
{
const char * env = getenv(name);
if (!env)
return;
what = parse<T>(env);
}
void ThreadFuzzer::initConfiguration()
{
#if OS_LINUX
num_cpus = get_nprocs();
#else
(void)num_cpus;
#endif
initFromEnv(cpu_time_period_us, "THREAD_FUZZER_CPU_TIME_PERIOD_US");
if (!cpu_time_period_us)
return;
initFromEnv(yield_probability, "THREAD_FUZZER_YIELD_PROBABILITY");
initFromEnv(migrate_probability, "THREAD_FUZZER_MIGRATE_PROBABILITY");
initFromEnv(sleep_probability, "THREAD_FUZZER_SLEEP_PROBABILITY");
initFromEnv(chaos_sleep_time_us, "THREAD_FUZZER_SLEEP_TIME_US");
}
void ThreadFuzzer::signalHandler(int)
{
auto saved_errno = errno;
auto & fuzzer = ThreadFuzzer::instance();
if (fuzzer.yield_probability > 0
&& std::bernoulli_distribution(fuzzer.yield_probability)(thread_local_rng))
{
sched_yield();
}
#if OS_LINUX
if (fuzzer.num_cpus > 0
&& fuzzer.migrate_probability > 0
&& std::bernoulli_distribution(fuzzer.migrate_probability)(thread_local_rng))
{
int migrate_to = std::uniform_int_distribution<>(0, fuzzer.num_cpus - 1)(thread_local_rng);
cpu_set_t set;
CPU_ZERO(&set);
CPU_SET(migrate_to, &set);
(void)sched_setaffinity(0, sizeof(set), &set);
}
#endif
if (fuzzer.sleep_probability > 0
&& fuzzer.chaos_sleep_time_us > 0
&& std::bernoulli_distribution(fuzzer.sleep_probability)(thread_local_rng))
{
sleepForNanoseconds(fuzzer.chaos_sleep_time_us * 1000);
}
errno = saved_errno;
}
void ThreadFuzzer::setup()
{
struct sigaction sa{};
sa.sa_handler = signalHandler;
sa.sa_flags = SA_RESTART;
if (sigemptyset(&sa.sa_mask))
throwFromErrno("Failed to clean signal mask for thread fuzzer", ErrorCodes::CANNOT_MANIPULATE_SIGSET);
if (sigaddset(&sa.sa_mask, SIGPROF))
throwFromErrno("Failed to add signal to mask for thread fuzzer", ErrorCodes::CANNOT_MANIPULATE_SIGSET);
if (sigaction(SIGPROF, &sa, nullptr))
throwFromErrno("Failed to setup signal handler for thread fuzzer", ErrorCodes::CANNOT_SET_SIGNAL_HANDLER);
static constexpr UInt32 TIMER_PRECISION = 1000000;
struct timeval interval;
interval.tv_sec = cpu_time_period_us / TIMER_PRECISION;
interval.tv_usec = cpu_time_period_us % TIMER_PRECISION;
struct itimerval timer = {.it_interval = interval, .it_value = interval};
if (0 != setitimer(ITIMER_PROF, &timer, nullptr))
throwFromErrno("Failed to create profiling timer", ErrorCodes::CANNOT_CREATE_TIMER);
}
}

View File

@ -0,0 +1,74 @@
#include <cstdint>
namespace DB
{
/** Allows to randomize thread scheduling and insert various glitches across whole program for testing purposes.
* It is done by setting up a timer that will send PROF signal to every thread when certain amount of CPU time has passed.
*
* To initialize ThreadFuzzer, call ThreadFuzzer::instance().
* The behaviour is controlled by environment variables:
*
* THREAD_FUZZER_CPU_TIME_PERIOD_US - period of signals in microseconds.
* THREAD_FUZZER_YIELD_PROBABILITY - probability to do 'sched_yield'.
* THREAD_FUZZER_MIGRATE_PROBABILITY - probability to set CPU affinity to random CPU core.
* THREAD_FUZZER_SLEEP_PROBABILITY - probability to sleep.
* THREAD_FUZZER_SLEEP_TIME_US - amount of time to sleep in microseconds.
*
* ThreadFuzzer will do nothing if environment variables are not set accordingly.
*
* The intention is to reproduce thread synchronization bugs (race conditions and deadlocks) more frequently in tests.
* We already have tests with TSan. But TSan only covers "physical" synchronization bugs, but not "logical" ones,
* where all data is protected by synchronization primitives, but we still have race conditions.
* Obviously, TSan cannot debug distributed synchronization bugs.
*
* The motivation for this tool is an evidence, that concurrency bugs are more likely to reproduce
* on bad unstable virtual machines in a dirty environments.
*
* The idea is not new, see also:
* https://channel9.msdn.com/blogs/peli/concurrency-fuzzing-with-cuzz
*
* Notes:
* - it can be also implemented with instrumentation (example: LLVM Xray) instead of signals.
* - it's also reasonable to insert glitches around interesting functions (example: mutex lock/unlock, starting of threads, etc.),
* it is doable with wrapping these functions (todo).
* - we should also make the sleep time random.
* - sleep obviously helps, but the effect of yield and migration is unclear.
*/
class ThreadFuzzer
{
public:
static ThreadFuzzer & instance()
{
static ThreadFuzzer res;
return res;
}
bool isEffective() const
{
return cpu_time_period_us != 0
&& (yield_probability > 0
|| migrate_probability > 0
|| (sleep_probability > 0 && chaos_sleep_time_us > 0));
}
private:
uint64_t cpu_time_period_us = 0;
double yield_probability = 0;
double migrate_probability = 0;
double sleep_probability = 0;
double chaos_sleep_time_us = 0;
int num_cpus = 0;
ThreadFuzzer();
void initConfiguration();
void setup();
static void signalHandler(int);
};
}

View File

@ -17,11 +17,6 @@
#include <fcntl.h>
namespace ProfileEvents
{
extern const Event QueryProfilerSignalOverruns;
}
namespace DB
{
@ -30,15 +25,13 @@ namespace
/// Normally query_id is a UUID (string with a fixed length) but user can provide custom query_id.
/// Thus upper bound on query_id length should be introduced to avoid buffer overflow in signal handler.
constexpr size_t QUERY_ID_MAX_LEN = 1024;
thread_local size_t write_trace_iteration = 0;
}
namespace ErrorCodes
{
}
LazyPipeFDs pipe;
TraceCollector::TraceCollector()
TraceCollector::TraceCollector(std::shared_ptr<TraceLog> trace_log_)
: trace_log(std::move(trace_log_))
{
pipe.open();
@ -51,38 +44,20 @@ TraceCollector::TraceCollector()
thread = ThreadFromGlobalPool(&TraceCollector::run, this);
}
TraceCollector::~TraceCollector()
{
if (!thread.joinable())
LOG_ERROR(&Poco::Logger::get("TraceCollector"), "TraceCollector thread is malformed and cannot be joined");
else
{
stop();
thread.join();
}
pipe.close();
}
void TraceCollector::collect(TraceType trace_type, const StackTrace & stack_trace, int overrun_count)
{
/// Quickly drop if signal handler is called too frequently.
/// Otherwise we may end up infinitelly processing signals instead of doing any useful work.
++write_trace_iteration;
if (overrun_count)
{
/// But pass with some frequency to avoid drop of all traces.
if (write_trace_iteration % overrun_count == 0)
{
ProfileEvents::increment(ProfileEvents::QueryProfilerSignalOverruns, overrun_count);
}
else
{
ProfileEvents::increment(ProfileEvents::QueryProfilerSignalOverruns, overrun_count + 1);
return;
}
}
void TraceCollector::collect(TraceType trace_type, const StackTrace & stack_trace, UInt64 size)
{
constexpr size_t buf_size = sizeof(char) + // TraceCollector stop flag
8 * sizeof(char) + // maximum VarUInt length for string size
QUERY_ID_MAX_LEN * sizeof(char) + // maximum query_id length
@ -99,7 +74,7 @@ void TraceCollector::collect(TraceType trace_type, const StackTrace & stack_trac
auto thread_id = CurrentThread::get().thread_id;
writeChar(false, out);
writeChar(false, out); /// true if requested to stop the collecting thread.
writeStringBinary(query_id, out);
size_t stack_trace_size = stack_trace.getSize();
@ -110,64 +85,27 @@ void TraceCollector::collect(TraceType trace_type, const StackTrace & stack_trac
writePODBinary(trace_type, out);
writePODBinary(thread_id, out);
writePODBinary(UInt64(0), out);
out.next();
}
void TraceCollector::collect(UInt64 size)
{
constexpr size_t buf_size = sizeof(char) + // TraceCollector stop flag
8 * sizeof(char) + // maximum VarUInt length for string size
QUERY_ID_MAX_LEN * sizeof(char) + // maximum query_id length
sizeof(UInt8) + // number of stack frames
sizeof(StackTrace::Frames) + // collected stack trace, maximum capacity
sizeof(TraceType) + // trace type
sizeof(UInt64) + // thread_id
sizeof(UInt64); // size
char buffer[buf_size];
WriteBufferFromFileDescriptorDiscardOnFailure out(pipe.fds_rw[1], buf_size, buffer);
StringRef query_id = CurrentThread::getQueryId();
query_id.size = std::min(query_id.size, QUERY_ID_MAX_LEN);
auto thread_id = CurrentThread::get().thread_id;
writeChar(false, out);
writeStringBinary(query_id, out);
const auto & stack_trace = StackTrace();
size_t stack_trace_size = stack_trace.getSize();
size_t stack_trace_offset = stack_trace.getOffset();
writeIntBinary(UInt8(stack_trace_size - stack_trace_offset), out);
for (size_t i = stack_trace_offset; i < stack_trace_size; ++i)
writePODBinary(stack_trace.getFrames()[i], out);
writePODBinary(TraceType::MEMORY, out);
writePODBinary(thread_id, out);
writePODBinary(size, out);
out.next();
}
/**
* Sends TraceCollector stop message
/** Sends TraceCollector stop message
*
* Each sequence of data for TraceCollector thread starts with a boolean flag.
* If this flag is true, TraceCollector must stop reading trace_pipe and exit.
* This function sends flag with a true value to stop TraceCollector gracefully.
*
* NOTE: TraceCollector will NOT stop immediately as there may be some data left in the pipe
* before stop message.
*/
void TraceCollector::stop()
{
WriteBufferFromFileDescriptor out(pipe.fds_rw[1]);
writeChar(true, out);
out.next();
thread.join();
}
void TraceCollector::run()
{
ReadBufferFromFileDescriptor in(pipe.fds_rw[0]);

View File

@ -15,28 +15,27 @@ namespace DB
class TraceLog;
enum class TraceType : UInt8
enum class TraceType : uint8_t
{
REAL_TIME,
CPU_TIME,
MEMORY,
Real,
CPU,
Memory,
};
class TraceCollector
{
public:
TraceCollector();
TraceCollector(std::shared_ptr<TraceLog> trace_log_);
~TraceCollector();
void setTraceLog(const std::shared_ptr<TraceLog> & trace_log_) { trace_log = trace_log_; }
void collect(TraceType type, const StackTrace & stack_trace, int overrun_count = 0);
void collect(UInt64 size);
/// Collect a stack trace. This method is signal safe.
/// Precondition: the TraceCollector object must be created.
/// size - for memory tracing is the amount of memory allocated; for other trace types it is 0.
static void collect(TraceType trace_type, const StackTrace & stack_trace, UInt64 size);
private:
std::shared_ptr<TraceLog> trace_log;
ThreadFromGlobalPool thread;
LazyPipeFDs pipe;
void run();
void stop();

View File

@ -109,7 +109,6 @@ struct UInt128TrivialHash
};
/** Used for aggregation, for putting a large number of constant-length keys in a hash table.
*/
struct UInt256

View File

@ -88,7 +88,6 @@ using namespace DB;
struct ZooKeeperRequest;
/** Usage scenario: look at the documentation for IKeeper class.
*/
class ZooKeeper : public IKeeper

View File

@ -200,4 +200,3 @@ TEST(zkutil, multi_create_sequential)
}

View File

@ -10,7 +10,6 @@
#endif
unsigned getNumberOfPhysicalCPUCores()
{
#if USE_CPUID

View File

@ -50,7 +50,6 @@ static bool parseNumber(const String & description, size_t l, size_t r, size_t &
}
/* Parse a string that generates shards and replicas. Separator - one of two characters | or ,
* depending on whether shards or replicas are generated.
* For example:

View File

@ -75,3 +75,6 @@ target_link_libraries (stopwatch PRIVATE clickhouse_common_io)
add_executable (symbol_index symbol_index.cpp)
target_link_libraries (symbol_index PRIVATE clickhouse_common_io)
add_executable (chaos_sanitizer chaos_sanitizer.cpp)
target_link_libraries (chaos_sanitizer PRIVATE clickhouse_common_io)

View File

@ -0,0 +1,56 @@
#include <thread>
#include <iostream>
#include <common/sleep.h>
#include <IO/ReadHelpers.h>
#include <Common/Exception.h>
#include <Common/ThreadFuzzer.h>
/** Prooves that ThreadFuzzer helps to find concurrency bugs.
*
* for i in {1..10}; do ./chaos_sanitizer 1000000; done
* for i in {1..10}; do THREAD_FUZZER_CPU_TIME_PERIOD_US=1000 THREAD_FUZZER_SLEEP_PROBABILITY=0.1 THREAD_FUZZER_SLEEP_TIME_US=100000 ./chaos_sanitizer 1000000; done
*/
int main(int argc, char ** argv)
{
const size_t num_iterations = argc >= 2 ? DB::parse<size_t>(argv[1]) : 1000000000;
std::cerr << (DB::ThreadFuzzer::instance().isEffective() ? "ThreadFuzzer is enabled.\n" : "ThreadFuzzer is not enabled.\n");
volatile size_t counter1 = 0;
volatile size_t counter2 = 0;
/// These threads are synchronized by sleep (that's intentionally incorrect).
std::thread t1([&]
{
for (size_t i = 0; i < num_iterations; ++i)
++counter1;
sleepForNanoseconds(100000000);
for (size_t i = 0; i < num_iterations; ++i)
++counter2;
});
std::thread t2([&]
{
for (size_t i = 0; i < num_iterations; ++i)
++counter2;
sleepForNanoseconds(100000000);
for (size_t i = 0; i < num_iterations; ++i)
++counter1;
});
t1.join();
t2.join();
std::cerr << "Result: " << counter1 << ", " << counter2 << "\n";
return 0;
}

View File

@ -7,7 +7,6 @@
#include <Common/HashTable/HashSet.h>
int main(int, char **)
{
{

View File

@ -22,7 +22,6 @@ using Key = UInt64;
using Value = UInt64;
/// Various hash functions to test
namespace Hashes
@ -336,7 +335,6 @@ static void NO_INLINE testForEachMapAndHash(const Key * data, size_t size)
}
int main(int argc, char ** argv)
{
if (argc < 2)

View File

@ -244,7 +244,6 @@ void aggregate5(Map & local_map, MapSmallLocks & global_map, Source::const_itera
}*/
int main(int argc, char ** argv)
{
size_t n = atoi(argv[1]);

View File

@ -283,7 +283,6 @@ struct Merger
};
int main(int argc, char ** argv)
{
size_t n = atoi(argv[1]);

View File

@ -6,7 +6,6 @@
#include <Common/HashTable/SmallTable.h>
int main(int, char **)
{
{

View File

@ -358,7 +358,6 @@ void Block::setColumns(const Columns & columns)
}
Block Block::cloneWithColumns(MutableColumns && columns) const
{
Block res;

View File

@ -12,7 +12,6 @@
#include <Core/ColumnsWithTypeAndName.h>
namespace DB
{

View File

@ -165,4 +165,13 @@ bool NamesAndTypesList::contains(const String & name) const
return false;
}
std::optional<NameAndTypePair> NamesAndTypesList::tryGetByName(const std::string & name) const
{
for (const NameAndTypePair & column : *this)
{
if (column.name == name)
return column;
}
return {};
}
}

View File

@ -73,7 +73,11 @@ public:
/// Unlike `filter`, returns columns in the order in which they go in `names`.
NamesAndTypesList addTypes(const Names & names) const;
/// Check that column contains in list
bool contains(const String & name) const;
/// Try to get column by name, return empty optional if column not found
std::optional<NameAndTypePair> tryGetByName(const std::string & name) const;
};
}

View File

@ -53,7 +53,7 @@ struct Settings : public SettingsCollection<Settings>
M(SettingUInt64, min_insert_block_size_rows, DEFAULT_INSERT_BLOCK_SIZE, "Squash blocks passed to INSERT query to specified size in rows, if blocks are not big enough.", 0) \
M(SettingUInt64, min_insert_block_size_bytes, (DEFAULT_INSERT_BLOCK_SIZE * 256), "Squash blocks passed to INSERT query to specified size in bytes, if blocks are not big enough.", 0) \
M(SettingUInt64, max_joined_block_size_rows, DEFAULT_BLOCK_SIZE, "Maximum block size for JOIN result (if join algorithm supports it). 0 means unlimited.", 0) \
M(SettingUInt64, max_insert_threads, 0, "The maximum number of threads to execute the INSERT SELECT query. By default, it is determined automatically.", 0) \
M(SettingUInt64, max_insert_threads, 0, "The maximum number of threads to execute the INSERT SELECT query. Values 0 or 1 means that INSERT SELECT is not run in parallel. Higher values will lead to higher memory usage. Parallel INSERT SELECT has effect only if the SELECT part is run on parallel, see 'max_threads' setting.", 0) \
M(SettingMaxThreads, max_threads, 0, "The maximum number of threads to execute the request. By default, it is determined automatically.", 0) \
M(SettingMaxThreads, max_alter_threads, 0, "The maximum number of threads to execute the ALTER requests. By default, it is determined automatically.", 0) \
M(SettingUInt64, max_read_buffer_size, DBMS_DEFAULT_BUFFER_SIZE, "The maximum size of the buffer to read from the filesystem.", 0) \
@ -333,7 +333,7 @@ struct Settings : public SettingsCollection<Settings>
M(SettingUInt64, max_memory_usage, 0, "Maximum memory usage for processing of single query. Zero means unlimited.", 0) \
M(SettingUInt64, max_memory_usage_for_user, 0, "Maximum memory usage for processing all concurrently running queries for the user. Zero means unlimited.", 0) \
M(SettingUInt64, max_memory_usage_for_all_queries, 0, "Maximum memory usage for processing all concurrently running queries on the server. Zero means unlimited.", 0) \
M(SettingUInt64, memory_profiler_step, 0, "Every number of bytes the memory profiler will dump the allocating stacktrace. Zero means disabled memory profiler.", 0) \
M(SettingUInt64, memory_profiler_step, 0, "Every number of bytes the memory profiler will collect the allocating stack trace. The minimal effective step is 4 MiB (less values will work as clamped to 4 MiB). Zero means disabled memory profiler.", 0) \
\
M(SettingUInt64, max_network_bandwidth, 0, "The maximum speed of data exchange over the network in bytes per second for a query. Zero means unlimited.", 0) \
M(SettingUInt64, max_network_bytes, 0, "The maximum number of bytes (compressed) to receive or transmit over the network for execution of the query.", 0) \

View File

@ -391,7 +391,6 @@ void SettingEnum<EnumType, Tag>::set(const Field & x)
}
String SettingURI::toString() const
{
return value.toString();

View File

@ -15,7 +15,6 @@ struct TypePair
};
template <typename T, bool _int, bool _float, bool _decimal, bool _datetime, typename F>
bool callOnBasicType(TypeIndex number, F && f)
{

View File

@ -1,7 +1,7 @@
#include <Common/typeid_cast.h>
#include <Functions/FunctionHelpers.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/evaluateMissingDefaults.h>
#include <Interpreters/inplaceBlockConversions.h>
#include <DataStreams/AddingDefaultsBlockInputStream.h>
#include <Columns/ColumnsNumber.h>

View File

@ -1,6 +1,6 @@
#include <DataStreams/TTLBlockInputStream.h>
#include <DataTypes/DataTypeDate.h>
#include <Interpreters/evaluateMissingDefaults.h>
#include <Interpreters/inplaceBlockConversions.h>
#include <Interpreters/SyntaxAnalyzer.h>
#include <Interpreters/ExpressionAnalyzer.h>

View File

@ -25,7 +25,6 @@ VersionedCollapsingSortedBlockInputStream::VersionedCollapsingSortedBlockInputSt
}
inline ALWAYS_INLINE static void writeRowSourcePart(WriteBuffer & buffer, RowSourcePart row_source)
{
if constexpr (sizeof(RowSourcePart) == 1)

View File

@ -66,7 +66,6 @@ DataTypeTuple::DataTypeTuple(const DataTypes & elems_, const Strings & names_)
}
std::string DataTypeTuple::doGetName() const
{
size_t size = elems.size();

View File

@ -42,7 +42,6 @@ std::string DataTypeDecimal<T>::doGetName() const
}
template <typename T>
bool DataTypeDecimal<T>::equals(const IDataType & rhs) const
{

View File

@ -27,7 +27,6 @@ namespace ErrorCodes
}
DatabaseLazy::DatabaseLazy(const String & name_, const String & metadata_path_, time_t expiration_time_, const Context & context_)
: DatabaseOnDisk(name_, metadata_path_, "DatabaseLazy (" + name_ + ")")
, expiration_time(expiration_time_)

View File

@ -25,7 +25,6 @@
#include <Poco/DirectoryIterator.h>
namespace DB
{

View File

@ -19,7 +19,6 @@
#include <Poco/DirectoryIterator.h>
namespace DB
{

View File

@ -7,6 +7,7 @@
#include <Common/quoteString.h>
#include <set>
#include <Poco/File.h>
@ -15,6 +16,7 @@ namespace DB
namespace ErrorCodes
{
extern const int BAD_ARGUMENTS;
extern const int EXCESSIVE_ELEMENT_IN_CONFIG;
extern const int UNKNOWN_DISK;
extern const int UNKNOWN_POLICY;
@ -48,7 +50,65 @@ DiskSelector::DiskSelector(const Poco::Util::AbstractConfiguration & config, con
}
const DiskPtr & DiskSelector::operator[](const String & name) const
DiskSelectorPtr DiskSelector::updateFromConfig(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, const Context & context) const
{
Poco::Util::AbstractConfiguration::Keys keys;
config.keys(config_prefix, keys);
auto & factory = DiskFactory::instance();
std::shared_ptr<DiskSelector> result = std::make_shared<DiskSelector>(*this);
std::set<String> old_disks_minus_new_disks;
for (const auto & [disk_name, _] : result->disks)
{
old_disks_minus_new_disks.insert(disk_name);
}
for (const auto & disk_name : keys)
{
if (!std::all_of(disk_name.begin(), disk_name.end(), isWordCharASCII))
throw Exception("Disk name can contain only alphanumeric and '_' (" + disk_name + ")", ErrorCodes::EXCESSIVE_ELEMENT_IN_CONFIG);
if (result->disks.count(disk_name) == 0)
{
auto disk_config_prefix = config_prefix + "." + disk_name;
result->disks.emplace(disk_name, factory.create(disk_name, config, disk_config_prefix, context));
}
else
{
old_disks_minus_new_disks.erase(disk_name);
/// TODO: Ideally ClickHouse shall complain if disk has changed, but
/// implementing that may appear as not trivial task.
}
}
if (!old_disks_minus_new_disks.empty())
{
WriteBufferFromOwnString warning;
if (old_disks_minus_new_disks.size() == 1)
writeString("Disk ", warning);
else
writeString("Disks ", warning);
int index = 0;
for (const String & name : old_disks_minus_new_disks)
{
if (index++ > 0)
writeString(", ", warning);
writeBackQuotedString(name, warning);
}
writeString(" disappeared from configuration, this change will be applied after restart of ClickHouse", warning);
LOG_WARNING(&Logger::get("DiskSelector"), warning.str());
}
return result;
}
DiskPtr DiskSelector::get(const String & name) const
{
auto it = disks.find(name);
if (it == disks.end())
@ -61,7 +121,7 @@ Volume::Volume(
String name_,
const Poco::Util::AbstractConfiguration & config,
const String & config_prefix,
const DiskSelector & disk_selector)
DiskSelectorPtr disk_selector)
: name(std::move(name_))
{
Poco::Util::AbstractConfiguration::Keys keys;
@ -74,7 +134,7 @@ Volume::Volume(
if (startsWith(disk, "disk"))
{
auto disk_name = config.getString(config_prefix + "." + disk);
disks.push_back(disk_selector[disk_name]);
disks.push_back(disk_selector->get(disk_name));
}
}
@ -162,7 +222,7 @@ StoragePolicy::StoragePolicy(
String name_,
const Poco::Util::AbstractConfiguration & config,
const String & config_prefix,
const DiskSelector & disks)
DiskSelectorPtr disks)
: name(std::move(name_))
{
String volumes_prefix = config_prefix + ".volumes";
@ -330,6 +390,28 @@ ReservationPtr StoragePolicy::makeEmptyReservationOnLargestDisk() const
}
void StoragePolicy::checkCompatibleWith(const StoragePolicyPtr & new_storage_policy) const
{
std::unordered_set<String> new_volume_names;
for (const auto & volume : new_storage_policy->getVolumes())
new_volume_names.insert(volume->getName());
for (const auto & volume : getVolumes())
{
if (new_volume_names.count(volume->getName()) == 0)
throw Exception("New storage policy shall contain volumes of old one", ErrorCodes::LOGICAL_ERROR);
std::unordered_set<String> new_disk_names;
for (const auto & disk : new_storage_policy->getVolumeByName(volume->getName())->disks)
new_disk_names.insert(disk->getName());
for (const auto & disk : volume->disks)
if (new_disk_names.count(disk->getName()) == 0)
throw Exception("New storage policy shall contain disks of old one", ErrorCodes::LOGICAL_ERROR);
}
}
size_t StoragePolicy::getVolumeIndexByDisk(const DiskPtr & disk_ptr) const
{
for (size_t i = 0; i < volumes.size(); ++i)
@ -346,7 +428,7 @@ size_t StoragePolicy::getVolumeIndexByDisk(const DiskPtr & disk_ptr) const
StoragePolicySelector::StoragePolicySelector(
const Poco::Util::AbstractConfiguration & config,
const String & config_prefix,
const DiskSelector & disks)
DiskSelectorPtr disks)
{
Poco::Util::AbstractConfiguration::Keys keys;
config.keys(config_prefix, keys);
@ -368,18 +450,39 @@ StoragePolicySelector::StoragePolicySelector(
/// Add default policy if it's not specified explicetly
if (policies.find(default_storage_policy_name) == policies.end())
{
auto default_volume = std::make_shared<Volume>(default_volume_name, std::vector<DiskPtr>{disks[default_disk_name]}, 0);
auto default_volume = std::make_shared<Volume>(default_volume_name, std::vector<DiskPtr>{disks->get(default_disk_name)}, 0);
auto default_policy = std::make_shared<StoragePolicy>(default_storage_policy_name, Volumes{default_volume}, 0.0);
policies.emplace(default_storage_policy_name, default_policy);
}
}
const StoragePolicyPtr & StoragePolicySelector::operator[](const String & name) const
StoragePolicySelectorPtr StoragePolicySelector::updateFromConfig(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, DiskSelectorPtr disks) const
{
Poco::Util::AbstractConfiguration::Keys keys;
config.keys(config_prefix, keys);
std::shared_ptr<StoragePolicySelector> result = std::make_shared<StoragePolicySelector>(config, config_prefix, disks);
for (const auto & [name, policy] : policies)
{
if (result->policies.count(name) == 0)
throw Exception("Storage policy " + backQuote(name) + " is missing in new configuration", ErrorCodes::BAD_ARGUMENTS);
policy->checkCompatibleWith(result->policies[name]);
}
return result;
}
StoragePolicyPtr StoragePolicySelector::get(const String & name) const
{
auto it = policies.find(name);
if (it == policies.end())
throw Exception("Unknown StoragePolicy " + name, ErrorCodes::UNKNOWN_POLICY);
return it->second;
}

View File

@ -17,15 +17,21 @@
namespace DB
{
class DiskSelector;
using DiskSelectorPtr = std::shared_ptr<const DiskSelector>;
/// Parse .xml configuration and store information about disks
/// Mostly used for introspection.
class DiskSelector
{
public:
DiskSelector(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, const Context & context);
DiskSelector(const DiskSelector & from): disks(from.disks) {}
DiskSelectorPtr updateFromConfig(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, const Context & context) const;
/// Get disk by name
const DiskPtr & operator[](const String & name) const;
DiskPtr get(const String & name) const;
/// Get all disks with names
const auto & getDisksMap() const { return disks; }
@ -54,7 +60,7 @@ public:
String name_,
const Poco::Util::AbstractConfiguration & config,
const String & config_prefix,
const DiskSelector & disk_selector);
DiskSelectorPtr disk_selector);
/// Next disk (round-robin)
///
@ -87,6 +93,8 @@ private:
using VolumePtr = std::shared_ptr<Volume>;
using Volumes = std::vector<VolumePtr>;
class StoragePolicy;
using StoragePolicyPtr = std::shared_ptr<const StoragePolicy>;
/**
* Contains all information about volumes configuration for Storage.
@ -95,7 +103,7 @@ using Volumes = std::vector<VolumePtr>;
class StoragePolicy
{
public:
StoragePolicy(String name_, const Poco::Util::AbstractConfiguration & config, const String & config_prefix, const DiskSelector & disks);
StoragePolicy(String name_, const Poco::Util::AbstractConfiguration & config, const String & config_prefix, DiskSelectorPtr disks);
StoragePolicy(String name_, Volumes volumes_, double move_factor_);
@ -146,6 +154,9 @@ public:
return getVolume(it->second);
}
/// Checks if storage policy can be replaced by another one.
void checkCompatibleWith(const StoragePolicyPtr & new_storage_policy) const;
private:
Volumes volumes;
const String name;
@ -158,17 +169,20 @@ private:
};
using StoragePolicyPtr = std::shared_ptr<const StoragePolicy>;
class StoragePolicySelector;
using StoragePolicySelectorPtr = std::shared_ptr<const StoragePolicySelector>;
/// Parse .xml configuration and store information about policies
/// Mostly used for introspection.
class StoragePolicySelector
{
public:
StoragePolicySelector(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, const DiskSelector & disks);
StoragePolicySelector(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, DiskSelectorPtr disks);
StoragePolicySelectorPtr updateFromConfig(const Poco::Util::AbstractConfiguration & config, const String & config_prefix, DiskSelectorPtr disks) const;
/// Policy by name
const StoragePolicyPtr & operator[](const String & name) const;
StoragePolicyPtr get(const String & name) const;
/// All policies
const std::map<String, StoragePolicyPtr> & getPoliciesMap() const { return policies; }

View File

@ -553,7 +553,6 @@ protected:
};
class ProtobufReader::ConverterFromString : public ConverterBaseImpl
{
public:
@ -864,7 +863,6 @@ PROTOBUF_READER_CREATE_CONVERTER_SPECIALIZATION_FOR_NUMBERS(google::protobuf::Fi
#undef PROTOBUF_READER_CREATE_CONVERTER_SPECIALIZATION_FOR_NUMBERS
class ProtobufReader::ConverterFromBool : public ConverterBaseImpl
{
public:

View File

@ -158,7 +158,6 @@ struct FixedStringOperationImpl
};
template <typename A, typename B, typename Op, typename ResultType = typename Op::ResultType>
struct BinaryOperationImpl : BinaryOperationImplBase<A, B, Op, ResultType>
{

View File

@ -18,7 +18,6 @@
#endif
/** Logical functions AND, OR, XOR and NOT support three-valued (or ternary) logic
* https://en.wikibooks.org/wiki/Structured_Query_Language/NULLs_and_the_Three_Valued_Logic
*

View File

@ -162,7 +162,6 @@ inline ALWAYS_INLINE void writeSlice(const NumericValueSlice<T> & slice, Generic
}
template <typename SourceA, typename SourceB, typename Sink>
void NO_INLINE concat(SourceA && src_a, SourceB && src_b, Sink && sink)
{

View File

@ -124,5 +124,3 @@ void registerFunctionCaseWithExpression(FunctionFactory & factory)
}

View File

@ -54,4 +54,3 @@ void registerFunctionGenerateUUIDv4(FunctionFactory & factory)
}

View File

@ -242,4 +242,3 @@ void registerFunctionMultiIf(FunctionFactory & factory)
}

View File

@ -152,4 +152,3 @@ void registerFunctionsReinterpretStringAs(FunctionFactory & factory)
}

View File

@ -18,4 +18,3 @@ void registerFunctionTimeSlot(FunctionFactory & factory)
}

View File

@ -62,4 +62,43 @@ void writeException(const Exception & e, WriteBuffer & buf, bool with_stack_trac
bool has_nested = false;
writeBinary(has_nested, buf);
}
/// The same, but quotes apply only if there are characters that do not match the identifier without quotes
template <typename F>
static inline void writeProbablyQuotedStringImpl(const StringRef & s, WriteBuffer & buf, F && write_quoted_string)
{
if (!s.size || !isValidIdentifierBegin(s.data[0]))
{
write_quoted_string(s, buf);
}
else
{
const char * pos = s.data + 1;
const char * end = s.data + s.size;
for (; pos < end; ++pos)
if (!isWordCharASCII(*pos))
break;
if (pos != end)
write_quoted_string(s, buf);
else
writeString(s, buf);
}
}
void writeProbablyBackQuotedString(const StringRef & s, WriteBuffer & buf)
{
writeProbablyQuotedStringImpl(s, buf, [](const StringRef & s_, WriteBuffer & buf_) { return writeBackQuotedString(s_, buf_); });
}
void writeProbablyDoubleQuotedString(const StringRef & s, WriteBuffer & buf)
{
writeProbablyQuotedStringImpl(s, buf, [](const StringRef & s_, WriteBuffer & buf_) { return writeDoubleQuotedString(s_, buf_); });
}
void writeProbablyBackQuotedStringMySQL(const StringRef & s, WriteBuffer & buf)
{
writeProbablyQuotedStringImpl(s, buf, [](const StringRef & s_, WriteBuffer & buf_) { return writeBackQuotedStringMySQL(s_, buf_); });
}
}

View File

@ -464,7 +464,6 @@ void writeAnyQuotedString(const char * begin, const char * end, WriteBuffer & bu
}
template <char quote_character>
void writeAnyQuotedString(const String & s, WriteBuffer & buf)
{
@ -510,40 +509,10 @@ inline void writeBackQuotedStringMySQL(const StringRef & s, WriteBuffer & buf)
}
/// The same, but quotes apply only if there are characters that do not match the identifier without quotes.
template <typename F>
inline void writeProbablyQuotedStringImpl(const StringRef & s, WriteBuffer & buf, F && write_quoted_string)
{
if (!s.size || !isValidIdentifierBegin(s.data[0]))
write_quoted_string(s, buf);
else
{
const char * pos = s.data + 1;
const char * end = s.data + s.size;
for (; pos < end; ++pos)
if (!isWordCharASCII(*pos))
break;
if (pos != end)
write_quoted_string(s, buf);
else
writeString(s, buf);
}
}
inline void writeProbablyBackQuotedString(const StringRef & s, WriteBuffer & buf)
{
writeProbablyQuotedStringImpl(s, buf, [](const StringRef & s_, WriteBuffer & buf_) { return writeBackQuotedString(s_, buf_); });
}
inline void writeProbablyDoubleQuotedString(const StringRef & s, WriteBuffer & buf)
{
writeProbablyQuotedStringImpl(s, buf, [](const StringRef & s_, WriteBuffer & buf_) { return writeDoubleQuotedString(s_, buf_); });
}
inline void writeProbablyBackQuotedStringMySQL(const StringRef & s, WriteBuffer & buf)
{
writeProbablyQuotedStringImpl(s, buf, [](const StringRef & s_, WriteBuffer & buf_) { return writeBackQuotedStringMySQL(s_, buf_); });
}
/// Write quoted if the string doesn't look like and identifier.
void writeProbablyBackQuotedString(const StringRef & s, WriteBuffer & buf);
void writeProbablyDoubleQuotedString(const StringRef & s, WriteBuffer & buf);
void writeProbablyBackQuotedStringMySQL(const StringRef & s, WriteBuffer & buf);
/** Outputs the string in for the CSV format.
@ -1020,7 +989,6 @@ void writeText(const std::vector<T> & x, WriteBuffer & buf)
}
/// Serialize exception (so that it can be transferred over the network)
void writeException(const Exception & e, WriteBuffer & buf, bool with_stack_trace);

View File

@ -61,5 +61,4 @@ using BloomFilterPtr = std::shared_ptr<BloomFilter>;
bool operator== (const BloomFilter & a, const BloomFilter & b);
}

View File

@ -56,7 +56,7 @@
#include <Common/TraceCollector.h>
#include <common/logger_useful.h>
#include <Common/RemoteHostFilter.h>
#include <ext/singleton.h>
namespace ProfileEvents
{
@ -151,9 +151,9 @@ struct ContextShared
/// Rules for selecting the compression settings, depending on the size of the part.
mutable std::unique_ptr<CompressionCodecSelector> compression_codec_selector;
/// Storage disk chooser for MergeTree engines
mutable std::unique_ptr<DiskSelector> merge_tree_disk_selector;
mutable std::shared_ptr<const DiskSelector> merge_tree_disk_selector;
/// Storage policy chooser for MergeTree engines
mutable std::unique_ptr<StoragePolicySelector> merge_tree_storage_policy_selector;
mutable std::shared_ptr<const StoragePolicySelector> merge_tree_storage_policy_selector;
std::optional<MergeTreeSettings> merge_tree_settings; /// Settings of MergeTree* engines.
std::atomic_size_t max_table_size_to_drop = 50000000000lu; /// Protects MergeTree tables from accidental DROP (50GB by default)
@ -164,6 +164,8 @@ struct ContextShared
RemoteHostFilter remote_host_filter; /// Allowed URL from config.xml
std::optional<TraceCollector> trace_collector; /// Thread collecting traces from threads executing queries
/// Named sessions. The user could specify session identifier to reuse settings and temporary tables in subsequent requests.
class SessionKeyHash
@ -294,15 +296,21 @@ struct ContextShared
schedule_pool.reset();
ddl_worker.reset();
ext::Singleton<TraceCollector>::reset();
/// Stop trace collector if any
trace_collector.reset();
}
bool hasTraceCollector() const
{
return trace_collector.has_value();
}
void initializeTraceCollector(std::shared_ptr<TraceLog> trace_log)
{
if (trace_log == nullptr)
if (hasTraceCollector())
return;
ext::Singleton<TraceCollector>()->setTraceLog(trace_log);
trace_collector.emplace(std::move(trace_log));
}
};
@ -569,7 +577,7 @@ VolumePtr Context::setTemporaryStorage(const String & path, const String & polic
}
else
{
StoragePolicyPtr tmp_policy = getStoragePolicySelector()[policy_name];
StoragePolicyPtr tmp_policy = getStoragePolicySelector()->get(policy_name);
if (tmp_policy->getVolumes().size() != 1)
throw Exception("Policy " + policy_name + " is used temporary files, such policy should have exactly one volume", ErrorCodes::NO_ELEMENTS_IN_CONFIG);
shared->tmp_volume = tmp_policy->getVolume(0);
@ -1786,6 +1794,11 @@ void Context::initializeTraceCollector()
shared->initializeTraceCollector(getTraceLog());
}
bool Context::hasTraceCollector() const
{
return shared->hasTraceCollector();
}
std::shared_ptr<QueryLog> Context::getQueryLog()
{
@ -1879,17 +1892,17 @@ CompressionCodecPtr Context::chooseCompressionCodec(size_t part_size, double par
}
const DiskPtr & Context::getDisk(const String & name) const
DiskPtr Context::getDisk(const String & name) const
{
auto lock = getLock();
const auto & disk_selector = getDiskSelector();
auto disk_selector = getDiskSelector();
return disk_selector[name];
return disk_selector->get(name);
}
DiskSelector & Context::getDiskSelector() const
DiskSelectorPtr Context::getDiskSelector() const
{
auto lock = getLock();
@ -1898,23 +1911,23 @@ DiskSelector & Context::getDiskSelector() const
constexpr auto config_name = "storage_configuration.disks";
auto & config = getConfigRef();
shared->merge_tree_disk_selector = std::make_unique<DiskSelector>(config, config_name, *this);
shared->merge_tree_disk_selector = std::make_shared<DiskSelector>(config, config_name, *this);
}
return *shared->merge_tree_disk_selector;
return shared->merge_tree_disk_selector;
}
const StoragePolicyPtr & Context::getStoragePolicy(const String & name) const
StoragePolicyPtr Context::getStoragePolicy(const String & name) const
{
auto lock = getLock();
auto & policy_selector = getStoragePolicySelector();
auto policy_selector = getStoragePolicySelector();
return policy_selector[name];
return policy_selector->get(name);
}
StoragePolicySelector & Context::getStoragePolicySelector() const
StoragePolicySelectorPtr Context::getStoragePolicySelector() const
{
auto lock = getLock();
@ -1923,9 +1936,30 @@ StoragePolicySelector & Context::getStoragePolicySelector() const
constexpr auto config_name = "storage_configuration.policies";
auto & config = getConfigRef();
shared->merge_tree_storage_policy_selector = std::make_unique<StoragePolicySelector>(config, config_name, getDiskSelector());
shared->merge_tree_storage_policy_selector = std::make_shared<StoragePolicySelector>(config, config_name, getDiskSelector());
}
return shared->merge_tree_storage_policy_selector;
}
void Context::updateStorageConfiguration(const Poco::Util::AbstractConfiguration & config)
{
auto lock = getLock();
if (shared->merge_tree_disk_selector)
shared->merge_tree_disk_selector = shared->merge_tree_disk_selector->updateFromConfig(config, "storage_configuration.disks", *this);
if (shared->merge_tree_storage_policy_selector)
{
try
{
shared->merge_tree_storage_policy_selector = shared->merge_tree_storage_policy_selector->updateFromConfig(config, "storage_configuration.policies", shared->merge_tree_disk_selector);
}
catch (Exception & e)
{
LOG_ERROR(shared->log, "An error has occured while reloading storage policies, storage policies were not applied: " << e.message());
}
}
return *shared->merge_tree_storage_policy_selector;
}

View File

@ -93,9 +93,11 @@ struct StorageID;
class IDisk;
using DiskPtr = std::shared_ptr<IDisk>;
class DiskSelector;
using DiskSelectorPtr = std::shared_ptr<const DiskSelector>;
class StoragePolicy;
using StoragePolicyPtr = std::shared_ptr<const StoragePolicy>;
class StoragePolicySelector;
using StoragePolicySelectorPtr = std::shared_ptr<const StoragePolicySelector>;
class IOutputFormat;
using OutputFormatPtr = std::shared_ptr<IOutputFormat>;
@ -516,8 +518,10 @@ public:
/// Call after initialization before using system logs. Call for global context.
void initializeSystemLogs();
/// Call after initialization before using trace collector.
void initializeTraceCollector();
bool hasTraceCollector();
bool hasTraceCollector() const;
/// Nullptr if the query log is not ready for this moment.
std::shared_ptr<QueryLog> getQueryLog();
@ -543,16 +547,18 @@ public:
/// Lets you select the compression codec according to the conditions described in the configuration file.
std::shared_ptr<ICompressionCodec> chooseCompressionCodec(size_t part_size, double part_size_ratio) const;
DiskSelector & getDiskSelector() const;
DiskSelectorPtr getDiskSelector() const;
/// Provides storage disks
const DiskPtr & getDisk(const String & name) const;
const DiskPtr & getDefaultDisk() const { return getDisk("default"); }
DiskPtr getDisk(const String & name) const;
DiskPtr getDefaultDisk() const { return getDisk("default"); }
StoragePolicySelector & getStoragePolicySelector() const;
StoragePolicySelectorPtr getStoragePolicySelector() const;
void updateStorageConfiguration(const Poco::Util::AbstractConfiguration & config);
/// Provides storage politics schemes
const StoragePolicyPtr & getStoragePolicy(const String &name) const;
StoragePolicyPtr getStoragePolicy(const String & name) const;
/// Get the server uptime in seconds.
time_t getUptimeSeconds() const;

View File

@ -553,7 +553,7 @@ bool DDLWorker::tryExecuteQuery(const String & query, const DDLTask & task, Exec
current_context = std::make_unique<Context>(context);
current_context->getClientInfo().query_kind = ClientInfo::QueryKind::SECONDARY_QUERY;
current_context->setCurrentQueryId(""); // generate random query_id
executeQuery(istr, ostr, false, *current_context, {}, {});
executeQuery(istr, ostr, false, *current_context, {});
}
catch (...)
{

View File

@ -48,11 +48,11 @@ struct TableWithColumnNames
, columns(columns_)
{}
void addHiddenColumns(const NamesAndTypesList & addition)
{
for (auto & column : addition)
hidden_columns.push_back(column.name);
}
TableWithColumnNames(const DatabaseAndTableWithAlias table_, Names && columns_, Names && hidden_columns_)
: table(table_)
, columns(columns_)
, hidden_columns(hidden_columns_)
{}
bool hasColumn(const String & name) const
{
@ -69,9 +69,42 @@ private:
mutable NameSet columns_set;
};
struct TableWithColumnNamesAndTypes
{
DatabaseAndTableWithAlias table;
NamesAndTypesList columns;
NamesAndTypesList hidden_columns;
TableWithColumnNamesAndTypes(const DatabaseAndTableWithAlias & table_, const NamesAndTypesList & columns_)
: table(table_)
, columns(columns_)
{}
void addHiddenColumns(const NamesAndTypesList & addition)
{
hidden_columns.insert(hidden_columns.end(), addition.begin(), addition.end());
}
TableWithColumnNames removeTypes() const
{
Names out_columns;
out_columns.reserve(columns.size());
for (auto & col : columns)
out_columns.push_back(col.name);
Names out_hidden_columns;
out_hidden_columns.reserve(hidden_columns.size());
for (auto & col : hidden_columns)
out_hidden_columns.push_back(col.name);
return TableWithColumnNames(table, std::move(out_columns), std::move(out_hidden_columns));
}
};
std::vector<DatabaseAndTableWithAlias> getDatabaseAndTables(const ASTSelectQuery & select_query, const String & current_database);
std::optional<DatabaseAndTableWithAlias> getDatabaseAndTable(const ASTSelectQuery & select, size_t table_number);
using TablesWithColumnNames = std::vector<TableWithColumnNames>;
using TablesWithColumnNamesAndTypes = std::vector<TableWithColumnNames>;
}

View File

@ -29,4 +29,3 @@ using InternalTextLogsQueuePtr = std::shared_ptr<InternalTextLogsQueue>;
}

View File

@ -117,7 +117,7 @@ BlockIO InterpreterInsertQuery::execute()
/// Passing 1 as subquery_depth will disable limiting size of intermediate result.
InterpreterSelectWithUnionQuery interpreter_select{query.select, context, SelectQueryOptions(QueryProcessingStage::Complete, 1)};
if (table->supportsParallelInsert() && settings.max_insert_threads > 0)
if (table->supportsParallelInsert() && settings.max_insert_threads > 1)
{
in_streams = interpreter_select.executeWithMultipleStreams(res.pipeline);
out_streams_size = std::min(size_t(settings.max_insert_threads), in_streams.size());

View File

@ -109,7 +109,6 @@ static QueryDescriptors extractQueriesExceptMeAndCheckAccess(const Block & proce
}
class SyncKillQueryInputStream : public IBlockInputStream
{
public:

View File

@ -56,7 +56,6 @@
#include <Storages/MergeTree/MergeTreeData.h>
#include <Storages/MergeTree/MergeTreeWhereOptimizer.h>
#include <Storages/IStorage.h>
#include <Storages/StorageValues.h>
#include <TableFunctions/ITableFunction.h>
#include <TableFunctions/TableFunctionFactory.h>
@ -152,7 +151,7 @@ String InterpreterSelectQuery::generateFilterActions(ExpressionActionsPtr & acti
table_expr->children.push_back(table_expr->database_and_table_name);
/// Using separate expression analyzer to prevent any possible alias injection
auto syntax_result = SyntaxAnalyzer(*context).analyzeSelect(query_ast, storage->getColumns().getAll());
auto syntax_result = SyntaxAnalyzer(*context).analyzeSelect(query_ast, SyntaxAnalyzerResult({}, storage));
SelectQueryExpressionAnalyzer analyzer(query_ast, syntax_result, *context);
actions = analyzer.simpleSelectActions();
@ -236,25 +235,22 @@ InterpreterSelectQuery::InterpreterSelectQuery(
throw Exception("Too deep subqueries. Maximum: " + settings.max_subquery_depth.toString(),
ErrorCodes::TOO_DEEP_SUBQUERIES);
CrossToInnerJoinVisitor::Data cross_to_inner;
CrossToInnerJoinVisitor(cross_to_inner).visit(query_ptr);
JoinedTables joined_tables(getSelectQuery());
if (joined_tables.hasJoins())
{
CrossToInnerJoinVisitor::Data cross_to_inner;
CrossToInnerJoinVisitor(cross_to_inner).visit(query_ptr);
JoinToSubqueryTransformVisitor::Data join_to_subs_data{*context};
JoinToSubqueryTransformVisitor(join_to_subs_data).visit(query_ptr);
JoinToSubqueryTransformVisitor::Data join_to_subs_data{*context};
JoinToSubqueryTransformVisitor(join_to_subs_data).visit(query_ptr);
joined_tables.reset(getSelectQuery());
}
max_streams = settings.max_threads;
auto & query = getSelectQuery();
ASTSelectQuery & query = getSelectQuery();
ASTPtr table_expression = extractTableExpression(query, 0);
String database_name, table_name;
bool is_table_func = false;
bool is_subquery = false;
if (table_expression)
{
is_table_func = table_expression->as<ASTFunction>();
is_subquery = table_expression->as<ASTSelectWithUnionQuery>();
}
const ASTPtr & left_table_expression = joined_tables.leftTableExpression();
if (input)
{
@ -266,60 +262,41 @@ InterpreterSelectQuery::InterpreterSelectQuery(
/// Read from prepared input.
source_header = input_pipe->getHeader();
}
else if (is_subquery)
else if (joined_tables.isLeftTableSubquery())
{
/// Read from subquery.
interpreter_subquery = std::make_unique<InterpreterSelectWithUnionQuery>(
table_expression, getSubqueryContext(*context), options.subquery(), required_columns);
left_table_expression, getSubqueryContext(*context), options.subquery());
source_header = interpreter_subquery->getSampleBlock();
}
else if (!storage)
{
if (is_table_func)
if (joined_tables.isLeftTableFunction())
{
/// Read from table function. propagate all settings from initSettings(),
/// alternative is to call on current `context`, but that can potentially pollute it.
storage = getSubqueryContext(*context).executeTableFunction(table_expression);
storage = getSubqueryContext(*context).executeTableFunction(left_table_expression);
}
else
{
getDatabaseAndTableNames(query, database_name, table_name, *context);
if (auto view_source = context->getViewSource())
{
auto & storage_values = static_cast<const StorageValues &>(*view_source);
auto tmp_table_id = storage_values.getStorageID();
if (tmp_table_id.database_name == database_name && tmp_table_id.table_name == table_name)
{
/// Read from view source.
storage = context->getViewSource();
}
}
if (!storage)
{
/// Read from table. Even without table expression (implicit SELECT ... FROM system.one).
storage = context->getTable(database_name, table_name);
}
}
storage = joined_tables.getLeftTableStorage(*context);
}
if (storage)
{
table_lock = storage->lockStructureForShare(false, context->getInitialQueryId());
table_id = storage->getStorageID();
}
/// Extract joined tables colunms if any.
/// It could get storage from context without lockStructureForShare(). TODO: add lock there or rewrite this logic.
JoinedTables joined_tables;
joined_tables.resolveTables(*query_ptr->as<ASTSelectQuery>(), storage, *context, source_header.getNamesAndTypesList());
joined_tables.resolveTables(getSubqueryContext(*context), storage);
}
else
joined_tables.resolveTables(getSubqueryContext(*context), source_header.getNamesAndTypesList());
auto analyze = [&] (bool try_move_to_prewhere = true)
{
syntax_analyzer_result = SyntaxAnalyzer(*context).analyzeSelect(
query_ptr, source_header.getNamesAndTypesList(), storage, options, joined_tables, required_result_column_names);
query_ptr, SyntaxAnalyzerResult(source_header.getNamesAndTypesList(), storage),
options, joined_tables.tablesWithColumns(), required_result_column_names);
/// Save scalar sub queries's results in the query context
if (context->hasQueryContext())
@ -353,12 +330,11 @@ InterpreterSelectQuery::InterpreterSelectQuery(
if (syntax_analyzer_result->rewrite_subqueries)
{
/// remake interpreter_subquery when PredicateOptimizer rewrites subqueries and main table is subquery
if (is_subquery)
if (joined_tables.isLeftTableSubquery())
interpreter_subquery = std::make_unique<InterpreterSelectWithUnionQuery>(
table_expression,
left_table_expression,
getSubqueryContext(*context),
options.subquery(),
required_columns);
options.subquery());
}
}
@ -430,6 +406,9 @@ InterpreterSelectQuery::InterpreterSelectQuery(
if (query.prewhere() && !query.where())
analysis_result.prewhere_info->need_filter = true;
const String & database_name = joined_tables.leftTableDatabase();
const String & table_name = joined_tables.leftTableName();
if (!table_name.empty() && !database_name.empty() /* always allow access to temporary tables */)
context->checkAccess(AccessType::SELECT, database_name, table_name, required_columns);
@ -447,25 +426,6 @@ InterpreterSelectQuery::InterpreterSelectQuery(
}
void InterpreterSelectQuery::getDatabaseAndTableNames(const ASTSelectQuery & query, String & database_name, String & table_name, const Context & context)
{
if (auto db_and_table = getDatabaseAndTable(query, 0))
{
table_name = db_and_table->table;
database_name = db_and_table->database;
/// If the database is not specified - use the current database.
if (database_name.empty() && !context.isExternalTableExist(table_name))
database_name = context.getCurrentDatabase();
}
else /// If the table is not specified - use the table `system.one`.
{
database_name = "system";
table_name = "one";
}
}
Block InterpreterSelectQuery::getSampleBlock()
{
return result_header;

View File

@ -152,10 +152,6 @@ private:
template <typename TPipeline>
void executeImpl(TPipeline & pipeline, const BlockInputStreamPtr & prepared_input, std::optional<Pipe> prepared_pipe, QueryPipeline & save_context_and_storage);
/** From which table to read. With JOIN, the "left" table is returned.
*/
static void getDatabaseAndTableNames(const ASTSelectQuery & query, String & database_name, String & table_name, const Context & context);
/// Different stages of query execution.
/// dry_run - don't read from table, use empty header block instead.

View File

@ -3,6 +3,9 @@
#include <Interpreters/getTableExpressions.h>
#include <Storages/IStorage.h>
#include <Storages/ColumnsDescription.h>
#include <Storages/StorageValues.h>
#include <Parsers/ASTFunction.h>
#include <Parsers/ASTSelectWithUnionQuery.h>
namespace DB
{
@ -15,61 +18,99 @@ namespace ErrorCodes
namespace
{
std::vector<TableWithColumnNames> getTablesWithColumns(const std::vector<const ASTTableExpression * > & table_expressions,
const Context & context)
template <typename T>
void checkTablesWithColumns(const std::vector<T> & tables_with_columns, const Context & context)
{
std::vector<TableWithColumnNames> tables_with_columns = getDatabaseAndTablesWithColumnNames(table_expressions, context);
auto & settings = context.getSettingsRef();
if (settings.joined_subquery_requires_alias && tables_with_columns.size() > 1)
{
for (auto & pr : tables_with_columns)
if (pr.table.table.empty() && pr.table.alias.empty())
for (auto & t : tables_with_columns)
if (t.table.table.empty() && t.table.alias.empty())
throw Exception("No alias for subquery or table function in JOIN (set joined_subquery_requires_alias=0 to disable restriction).",
ErrorCodes::ALIAS_REQUIRED);
}
return tables_with_columns;
}
}
JoinedTables::JoinedTables(const ASTSelectQuery & select_query)
: table_expressions(getTableExpressions(select_query))
, left_table_expression(extractTableExpression(select_query, 0))
, left_db_and_table(getDatabaseAndTable(select_query, 0))
{}
void JoinedTables::resolveTables(const ASTSelectQuery & select_query, StoragePtr storage, const Context & context,
const NamesAndTypesList & source_columns)
bool JoinedTables::isLeftTableSubquery() const
{
if (!storage)
return left_table_expression && left_table_expression->as<ASTSelectWithUnionQuery>();
}
bool JoinedTables::isLeftTableFunction() const
{
return left_table_expression && left_table_expression->as<ASTFunction>();
}
StoragePtr JoinedTables::getLeftTableStorage(Context & context)
{
StoragePtr storage;
if (left_db_and_table)
{
if (auto db_and_table = getDatabaseAndTable(select_query, 0))
storage = context.tryGetTable(db_and_table->database, db_and_table->table);
database_name = left_db_and_table->database;
table_name = left_db_and_table->table;
/// If the database is not specified - use the current database.
if (database_name.empty() && !context.isExternalTableExist(table_name))
database_name = context.getCurrentDatabase();
}
else /// If the table is not specified - use the table `system.one`.
{
database_name = "system";
table_name = "one";
}
std::vector<const ASTTableExpression *> table_expressions = getTableExpressions(select_query);
tables_with_columns = getTablesWithColumns(table_expressions, context);
if (auto view_source = context.getViewSource())
{
auto & storage_values = static_cast<const StorageValues &>(*view_source);
auto tmp_table_id = storage_values.getStorageID();
if (tmp_table_id.database_name == database_name && tmp_table_id.table_name == table_name)
{
/// Read from view source.
storage = context.getViewSource();
}
}
if (!storage)
{
/// Read from table. Even without table expression (implicit SELECT ... FROM system.one).
storage = context.getTable(database_name, table_name);
}
return storage;
}
void JoinedTables::resolveTables(const Context & context, StoragePtr storage)
{
tables_with_columns = getDatabaseAndTablesWithColumns(table_expressions, context);
checkTablesWithColumns(tables_with_columns, context);
if (tables_with_columns.empty())
{
if (storage)
{
const ColumnsDescription & storage_columns = storage->getColumns();
tables_with_columns.emplace_back(DatabaseAndTableWithAlias{}, storage_columns.getOrdinary().getNames());
auto & table = tables_with_columns.back();
table.addHiddenColumns(storage_columns.getMaterialized());
table.addHiddenColumns(storage_columns.getAliases());
table.addHiddenColumns(storage_columns.getVirtuals());
}
else
{
Names columns;
columns.reserve(source_columns.size());
for (const auto & column : source_columns)
columns.push_back(column.name);
tables_with_columns.emplace_back(DatabaseAndTableWithAlias{}, columns);
}
const ColumnsDescription & storage_columns = storage->getColumns();
tables_with_columns.emplace_back(DatabaseAndTableWithAlias{}, storage_columns.getOrdinary());
auto & table = tables_with_columns.back();
table.addHiddenColumns(storage_columns.getMaterialized());
table.addHiddenColumns(storage_columns.getAliases());
table.addHiddenColumns(storage_columns.getVirtuals());
}
}
if (table_expressions.size() > 1)
columns_from_joined_table = getColumnsFromTableExpression(*table_expressions[1], context);
void JoinedTables::resolveTables(const Context & context, const NamesAndTypesList & source_columns)
{
tables_with_columns = getDatabaseAndTablesWithColumns(table_expressions, context);
checkTablesWithColumns(tables_with_columns, context);
if (tables_with_columns.empty())
tables_with_columns.emplace_back(DatabaseAndTableWithAlias{}, source_columns);
}
}

View File

@ -16,15 +16,42 @@ class Context;
class JoinedTables
{
public:
void resolveTables(const ASTSelectQuery & select_query, StoragePtr storage, const Context & context,
const NamesAndTypesList & source_columns);
JoinedTables() = default;
JoinedTables(const ASTSelectQuery & select_query);
const std::vector<TableWithColumnNames> & tablesWithColumns() const { return tables_with_columns; }
const NamesAndTypesList & secondTableColumns() const { return columns_from_joined_table; }
void reset(const ASTSelectQuery & select_query)
{
*this = JoinedTables(select_query);
}
StoragePtr getLeftTableStorage(Context & context);
/// Resolve columns or get from storage. It assumes storage is not nullptr.
void resolveTables(const Context & context, StoragePtr storage);
/// Resolve columns or get from source list.
void resolveTables(const Context & context, const NamesAndTypesList & source_columns);
const std::vector<TableWithColumnNamesAndTypes> & tablesWithColumns() const { return tables_with_columns; }
bool isLeftTableSubquery() const;
bool isLeftTableFunction() const;
bool hasJoins() const { return table_expressions.size() > 1; }
const ASTPtr & leftTableExpression() const { return left_table_expression; }
const String & leftTableDatabase() const { return database_name; }
const String & leftTableName() const { return table_name; }
private:
std::vector<TableWithColumnNames> tables_with_columns;
NamesAndTypesList columns_from_joined_table;
std::vector<const ASTTableExpression *> table_expressions;
std::vector<TableWithColumnNamesAndTypes> tables_with_columns;
/// Legacy (duplicated left table values)
ASTPtr left_table_expression;
std::optional<DatabaseAndTableWithAlias> left_db_and_table;
/// left_db_and_table or 'system.one'
String database_name;
String table_name;
};
}

View File

@ -284,6 +284,7 @@ ASTPtr MutationsInterpreter::prepare(bool dry_run)
if (commands.empty())
throw Exception("Empty mutation commands list", ErrorCodes::LOGICAL_ERROR);
const ColumnsDescription & columns_desc = storage->getColumns();
const IndicesDescription & indices_desc = storage->getIndices();
NamesAndTypesList all_columns = columns_desc.getAllPhysical();
@ -292,7 +293,9 @@ ASTPtr MutationsInterpreter::prepare(bool dry_run)
for (const MutationCommand & command : commands)
{
for (const auto & kv : command.column_to_update_expression)
{
updated_columns.insert(kv.first);
}
}
/// We need to know which columns affect which MATERIALIZED columns and data skipping indices
@ -436,6 +439,15 @@ ASTPtr MutationsInterpreter::prepare(bool dry_run)
}
}
}
else if (command.type == MutationCommand::READ_COLUMN)
{
if (stages.empty() || !stages.back().column_to_updated.empty())
stages.emplace_back(context);
if (stages.size() == 1) /// First stage only supports filtering and can't update columns.
stages.emplace_back(context);
stages.back().column_to_updated.emplace(command.column_name, std::make_shared<ASTIdentifier>(command.column_name));
}
else
throw Exception("Unknown mutation command type: " + DB::toString<int>(command.type), ErrorCodes::UNKNOWN_MUTATION_COMMAND);
}
@ -506,6 +518,7 @@ ASTPtr MutationsInterpreter::prepareInterpreterSelectQuery(std::vector<Stage> &
{
NamesAndTypesList all_columns = storage->getColumns().getAllPhysical();
/// Next, for each stage calculate columns changed by this and previous stages.
for (size_t i = 0; i < prepared_stages.size(); ++i)
{
@ -518,8 +531,6 @@ ASTPtr MutationsInterpreter::prepareInterpreterSelectQuery(std::vector<Stage> &
if (i > 0)
prepared_stages[i].output_columns = prepared_stages[i - 1].output_columns;
else if (!commands.additional_columns.empty())
prepared_stages[i].output_columns.insert(commands.additional_columns.begin(), commands.additional_columns.end());
if (prepared_stages[i].output_columns.size() < all_columns.size())
{

View File

@ -197,8 +197,14 @@ ProcessList::EntryPtr ProcessList::insert(const String & query_, const IAST * as
/// Set query-level memory trackers
thread_group->memory_tracker.setOrRaiseHardLimit(process_it->max_memory_usage);
thread_group->memory_tracker.setOrRaiseProfilerLimit(settings.memory_profiler_step);
thread_group->memory_tracker.setProfilerStep(settings.memory_profiler_step);
if (query_context.hasTraceCollector())
{
/// Set up memory profiling
thread_group->memory_tracker.setOrRaiseProfilerLimit(settings.memory_profiler_step);
thread_group->memory_tracker.setProfilerStep(settings.memory_profiler_step);
}
thread_group->memory_tracker.setDescription("(for query)");
if (process_it->memory_tracker_fault_probability)
thread_group->memory_tracker.setFaultProbability(process_it->memory_tracker_fault_probability);

View File

@ -56,4 +56,3 @@ class QueryThreadLog : public SystemLog<QueryThreadLogElement>
}

View File

@ -458,9 +458,19 @@ MergeTreeSetIndex::MergeTreeSetIndex(const Columns & set_elements, std::vector<K
size_t tuple_size = indexes_mapping.size();
ordered_set.resize(tuple_size);
/// Create columns for points here to avoid extra allocations at 'checkInRange'.
left_point.reserve(tuple_size);
right_point.reserve(tuple_size);
for (size_t i = 0; i < tuple_size; ++i)
{
ordered_set[i] = set_elements[indexes_mapping[i].tuple_index];
left_point.emplace_back(ordered_set[i]->cloneEmpty());
right_point.emplace_back(ordered_set[i]->cloneEmpty());
}
Block block_to_sort;
SortDescription sort_description;
for (size_t i = 0; i < tuple_size; ++i)
@ -484,13 +494,6 @@ BoolMask MergeTreeSetIndex::checkInRange(const std::vector<Range> & key_ranges,
{
size_t tuple_size = indexes_mapping.size();
using FieldWithInfinityTuple = std::vector<FieldWithInfinity>;
FieldWithInfinityTuple left_point;
FieldWithInfinityTuple right_point;
left_point.reserve(tuple_size);
right_point.reserve(tuple_size);
bool invert_left_infinities = false;
bool invert_right_infinities = false;
@ -512,14 +515,14 @@ BoolMask MergeTreeSetIndex::checkInRange(const std::vector<Range> & key_ranges,
if (!new_range->left_included)
invert_left_infinities = true;
left_point.push_back(FieldWithInfinity(new_range->left));
left_point[i].update(new_range->left);
}
else
{
if (invert_left_infinities)
left_point.push_back(FieldWithInfinity::getPlusinfinity());
left_point[i].update(ValueWithInfinity::PLUS_INFINITY);
else
left_point.push_back(FieldWithInfinity::getMinusInfinity());
left_point[i].update(ValueWithInfinity::MINUS_INFINITY);
}
if (new_range->right_bounded)
@ -527,51 +530,78 @@ BoolMask MergeTreeSetIndex::checkInRange(const std::vector<Range> & key_ranges,
if (!new_range->right_included)
invert_right_infinities = true;
right_point.push_back(FieldWithInfinity(new_range->right));
right_point[i].update(new_range->right);
}
else
{
if (invert_right_infinities)
right_point.push_back(FieldWithInfinity::getMinusInfinity());
right_point[i].update(ValueWithInfinity::MINUS_INFINITY);
else
right_point.push_back(FieldWithInfinity::getPlusinfinity());
right_point[i].update(ValueWithInfinity::PLUS_INFINITY);
}
}
/// This allows to construct tuple in 'ordered_set' at specified index for comparison with range.
auto indices = ext::range(0, ordered_set.at(0)->size());
auto extract_tuple = [tuple_size, this](size_t i)
auto compare = [](const IColumn & lhs, const ValueWithInfinity & rhs, size_t row)
{
/// Inefficient.
FieldWithInfinityTuple res;
res.reserve(tuple_size);
for (size_t j = 0; j < tuple_size; ++j)
res.emplace_back((*ordered_set[j])[i]);
return res;
auto type = rhs.getType();
/// Return inverted infinity sign, because in 'lhs' all values are finite.
if (type != ValueWithInfinity::NORMAL)
return -static_cast<int>(type);
return lhs.compareAt(row, 0, rhs.getColumnIfFinite(), 1);
};
auto compare = [&extract_tuple](size_t i, const FieldWithInfinityTuple & rhs)
auto less = [this, &compare, tuple_size](size_t row, const auto & point)
{
return extract_tuple(i) < rhs;
for (size_t i = 0; i < tuple_size; ++i)
{
int res = compare(*ordered_set[i], point[i], row);
if (res)
return res < 0;
}
return false;
};
auto equals = [this, &compare, tuple_size](size_t row, const auto & point)
{
for (size_t i = 0; i < tuple_size; ++i)
if (compare(*ordered_set[i], point[i], row) != 0)
return false;
return true;
};
/** Because each parallelogram maps to a contiguous sequence of elements
* layed out in the lexicographically increasing order, the set intersects the range
* if and only if either bound coincides with an element or at least one element
* is between the lower bounds
*/
auto left_lower = std::lower_bound(indices.begin(), indices.end(), left_point, compare);
auto right_lower = std::lower_bound(indices.begin(), indices.end(), right_point, compare);
* layed out in the lexicographically increasing order, the set intersects the range
* if and only if either bound coincides with an element or at least one element
* is between the lower bounds
*/
auto indices = ext::range(0, size());
auto left_lower = std::lower_bound(indices.begin(), indices.end(), left_point, less);
auto right_lower = std::lower_bound(indices.begin(), indices.end(), right_point, less);
return
{
left_lower != right_lower
|| (left_lower != indices.end() && extract_tuple(*left_lower) == left_point)
|| (right_lower != indices.end() && extract_tuple(*right_lower) == right_point),
|| (left_lower != indices.end() && equals(*left_lower, left_point))
|| (right_lower != indices.end() && equals(*right_lower, right_point)),
true
};
}
void ValueWithInfinity::update(const Field & x)
{
/// Keep at most one element in column.
if (!column->empty())
column->popBack(1);
column->insert(x);
type = NORMAL;
}
const IColumn & ValueWithInfinity::getColumnIfFinite() const
{
if (type != NORMAL)
throw Exception("Trying to get column of infinite type", ErrorCodes::LOGICAL_ERROR);
return *column;
}
}

View File

@ -16,7 +16,6 @@ namespace DB
{
struct Range;
class FieldWithInfinity;
class IFunctionBase;
using FunctionBasePtr = std::shared_ptr<IFunctionBase>;
@ -180,6 +179,36 @@ using Sets = std::vector<SetPtr>;
class IFunction;
using FunctionPtr = std::shared_ptr<IFunction>;
/** Class that represents single value with possible infinities.
* Single field is stored in column for more optimal inplace comparisons with other regular columns.
* Extracting fields from columns and further their comparison is suboptimal and requires extra copying.
*/
class ValueWithInfinity
{
public:
enum Type
{
MINUS_INFINITY = -1,
NORMAL = 0,
PLUS_INFINITY = 1
};
ValueWithInfinity(MutableColumnPtr && column_)
: column(std::move(column_)), type(NORMAL) {}
void update(const Field & x);
void update(Type type_) { type = type_; }
const IColumn & getColumnIfFinite() const;
Type getType() const { return type; }
private:
MutableColumnPtr column;
Type type;
};
/// Class for checkInRange function.
class MergeTreeSetIndex
{
@ -203,6 +232,11 @@ public:
private:
Columns ordered_set;
std::vector<KeyTuplePositionMapping> indexes_mapping;
using ColumnsWithInfinity = std::vector<ValueWithInfinity>;
ColumnsWithInfinity left_point;
ColumnsWithInfinity right_point;
};
}

View File

@ -82,24 +82,6 @@ using CustomizeFunctionsMatcher = OneTypeMatcher<CustomizeFunctionsData>;
using CustomizeFunctionsVisitor = InDepthNodeVisitor<CustomizeFunctionsMatcher, true>;
/// Add columns from storage to source_columns list.
void collectSourceColumns(const ColumnsDescription & columns, NamesAndTypesList & source_columns, bool add_virtuals)
{
auto physical_columns = columns.getAllPhysical();
if (source_columns.empty())
source_columns.swap(physical_columns);
else
source_columns.insert(source_columns.end(), physical_columns.begin(), physical_columns.end());
if (add_virtuals)
{
const auto & storage_aliases = columns.getAliases();
const auto & storage_virtuals = columns.getVirtuals();
source_columns.insert(source_columns.end(), storage_aliases.begin(), storage_aliases.end());
source_columns.insert(source_columns.end(), storage_virtuals.begin(), storage_virtuals.end());
}
}
/// Translate qualified names such as db.table.column, table.column, table_alias.column to names' normal form.
/// Expand asterisks and qualified asterisks with column names.
/// There would be columns in normal form & column aliases after translation. Column & column alias would be normalized in QueryNormalizer.
@ -616,20 +598,32 @@ std::vector<const ASTFunction *> getAggregates(ASTPtr & query, const ASTSelectQu
}
/// Add columns from storage to source_columns list. Deduplicate resulted list.
void SyntaxAnalyzerResult::collectSourceColumns(bool add_virtuals)
{
if (storage)
{
const ColumnsDescription & columns = storage->getColumns();
auto columns_from_storage = add_virtuals ? columns.getAll() : columns.getAllPhysical();
if (source_columns.empty())
source_columns.swap(columns_from_storage);
else
source_columns.insert(source_columns.end(), columns_from_storage.begin(), columns_from_storage.end());
}
source_columns_set = removeDuplicateColumns(source_columns);
}
/// Calculate which columns are required to execute the expression.
/// Then, delete all other columns from the list of available columns.
/// After execution, columns will only contain the list of columns needed to read from the table.
void SyntaxAnalyzerResult::collectUsedColumns(const ASTPtr & query, const NamesAndTypesList & additional_source_columns)
void SyntaxAnalyzerResult::collectUsedColumns(const ASTPtr & query)
{
/// We calculate required_source_columns with source_columns modifications and swap them on exit
required_source_columns = source_columns;
if (!additional_source_columns.empty())
{
source_columns.insert(source_columns.end(), additional_source_columns.begin(), additional_source_columns.end());
removeDuplicateColumns(source_columns);
}
RequiredSourceColumnsVisitor::Data columns_context;
RequiredSourceColumnsVisitor(columns_context).visit(query);
@ -787,10 +781,9 @@ void SyntaxAnalyzerResult::collectUsedColumns(const ASTPtr & query, const NamesA
SyntaxAnalyzerResultPtr SyntaxAnalyzer::analyzeSelect(
ASTPtr & query,
const NamesAndTypesList & source_columns,
StoragePtr storage,
SyntaxAnalyzerResult && result,
const SelectQueryOptions & select_options,
const JoinedTables & joined_tables,
const std::vector<TableWithColumnNamesAndTypes> & tables_with_columns,
const Names & required_result_columns) const
{
auto * select_query = query->as<ASTSelectQuery>();
@ -802,31 +795,28 @@ SyntaxAnalyzerResultPtr SyntaxAnalyzer::analyzeSelect(
const auto & settings = context.getSettingsRef();
SyntaxAnalyzerResult result;
result.storage = storage;
result.source_columns = source_columns;
const NameSet & source_columns_set = result.source_columns_set;
result.analyzed_join = std::make_shared<AnalyzedJoin>(settings, context.getTemporaryVolume());
if (storage)
collectSourceColumns(storage->getColumns(), result.source_columns, true);
NameSet source_columns_set = removeDuplicateColumns(result.source_columns);
if (remove_duplicates)
renameDuplicatedColumns(select_query);
if (settings.enable_optimize_predicate_expression)
replaceJoinedTable(*select_query);
const std::vector<TableWithColumnNames> & tables_with_columns = joined_tables.tablesWithColumns();
result.analyzed_join->columns_from_joined_table = joined_tables.secondTableColumns();
/// TODO: Remove unneeded conversion
std::vector<TableWithColumnNames> tables_with_column_names;
for (const auto & table : tables_with_columns)
tables_with_column_names.emplace_back(table.removeTypes());
if (result.analyzed_join->columns_from_joined_table.size())
if (tables_with_columns.size() > 1)
{
result.analyzed_join->columns_from_joined_table = tables_with_columns[1].columns;
result.analyzed_join->deduplicateAndQualifyColumnNames(
source_columns_set, tables_with_columns[1].table.getQualifiedNamePrefix());
}
translateQualifiedNames(query, *select_query, source_columns_set, tables_with_columns);
translateQualifiedNames(query, *select_query, source_columns_set, tables_with_column_names);
/// Rewrite IN and/or JOIN for distributed tables according to distributed_product_mode setting.
InJoinSubqueriesPreprocessor(context).visit(query);
@ -849,7 +839,7 @@ SyntaxAnalyzerResultPtr SyntaxAnalyzer::analyzeSelect(
optimizeIf(query, result.aliases, settings.optimize_if_chain_to_miltiif);
/// Push the predicate expression down to the subqueries.
result.rewrite_subqueries = PredicateExpressionsOptimizer(context, tables_with_columns, settings).optimize(*select_query);
result.rewrite_subqueries = PredicateExpressionsOptimizer(context, tables_with_column_names, settings).optimize(*select_query);
/// GROUP BY injective function elimination.
optimizeGroupBy(select_query, source_columns_set, context);
@ -868,11 +858,11 @@ SyntaxAnalyzerResultPtr SyntaxAnalyzer::analyzeSelect(
setJoinStrictness(*select_query, settings.join_default_strictness, settings.any_join_distinct_right_table_keys,
result.analyzed_join->table_join);
collectJoinedColumns(*result.analyzed_join, *select_query, tables_with_columns, result.aliases);
collectJoinedColumns(*result.analyzed_join, *select_query, tables_with_column_names, result.aliases);
}
result.aggregates = getAggregates(query, *select_query);
result.collectUsedColumns(query, {});
result.collectUsedColumns(query);
return std::make_shared<const SyntaxAnalyzerResult>(result);
}
@ -883,13 +873,7 @@ SyntaxAnalyzerResultPtr SyntaxAnalyzer::analyze(ASTPtr & query, const NamesAndTy
const auto & settings = context.getSettingsRef();
SyntaxAnalyzerResult result;
result.storage = storage;
result.source_columns = source_columns;
if (storage)
collectSourceColumns(storage->getColumns(), result.source_columns, false);
removeDuplicateColumns(result.source_columns);
SyntaxAnalyzerResult result(source_columns, storage, false);
normalize(query, result.aliases, settings);
@ -899,7 +883,7 @@ SyntaxAnalyzerResultPtr SyntaxAnalyzer::analyze(ASTPtr & query, const NamesAndTy
optimizeIf(query, result.aliases, settings.optimize_if_chain_to_miltiif);
assertNoAggregates(query, "in wrong place");
result.collectUsedColumns(query, {});
result.collectUsedColumns(query);
return std::make_shared<const SyntaxAnalyzerResult>(result);
}

View File

@ -4,7 +4,7 @@
#include <Core/NamesAndTypes.h>
#include <Interpreters/Aliases.h>
#include <Interpreters/SelectQueryOptions.h>
#include <Interpreters/JoinedTables.h>
#include <Interpreters/DatabaseAndTableWithAlias.h>
#include <Storages/IStorage_fwd.h>
namespace DB
@ -23,6 +23,7 @@ struct SyntaxAnalyzerResult
std::shared_ptr<AnalyzedJoin> analyzed_join;
NamesAndTypesList source_columns;
NameSet source_columns_set; /// Set of names of source_columns.
/// Set of columns that are enough to read from the table to evaluate the expression. It does not include joined columns.
NamesAndTypesList required_source_columns;
@ -50,7 +51,15 @@ struct SyntaxAnalyzerResult
bool maybe_optimize_trivial_count = false;
void collectUsedColumns(const ASTPtr & query, const NamesAndTypesList & additional_source_columns);
SyntaxAnalyzerResult(const NamesAndTypesList & source_columns_, StoragePtr storage_ = {}, bool add_virtuals = true)
: storage(storage_)
, source_columns(source_columns_)
{
collectSourceColumns(add_virtuals);
}
void collectSourceColumns(bool add_virtuals);
void collectUsedColumns(const ASTPtr & query);
Names requiredSourceColumns() const { return required_source_columns.getNames(); }
const Scalars & getScalars() const { return scalars; }
};
@ -82,10 +91,9 @@ public:
/// Analyze and rewrite select query
SyntaxAnalyzerResultPtr analyzeSelect(
ASTPtr & query,
const NamesAndTypesList & source_columns,
StoragePtr storage = {},
SyntaxAnalyzerResult && result,
const SelectQueryOptions & select_options = {},
const JoinedTables & joined_tables = {},
const std::vector<TableWithColumnNamesAndTypes> & tables_with_columns = {},
const Names & required_result_columns = {}) const;
private:

View File

@ -9,8 +9,6 @@
#include <Common/ThreadProfileEvents.h>
#include <Common/TraceCollector.h>
#include <ext/singleton.h>
#if defined(OS_LINUX)
# include <Common/hasLinuxCapability.h>
@ -157,7 +155,7 @@ void ThreadStatus::finalizePerformanceCounters()
void ThreadStatus::initQueryProfiler()
{
/// query profilers are useless without trace collector
if (!global_context || !ext::Singleton<TraceCollector>::isInitialized())
if (!global_context || !global_context->hasTraceCollector())
return;
const auto & settings = query_context->getSettingsRef();

View File

@ -11,10 +11,11 @@ using namespace DB;
using TraceDataType = TraceLogElement::TraceDataType;
const TraceDataType::Values TraceLogElement::trace_values = {
{"Real", static_cast<UInt8>(TraceType::REAL_TIME)},
{"CPU", static_cast<UInt8>(TraceType::CPU_TIME)},
{"Memory", static_cast<UInt8>(TraceType::MEMORY)},
const TraceDataType::Values TraceLogElement::trace_values =
{
{"Real", static_cast<UInt8>(TraceType::Real)},
{"CPU", static_cast<UInt8>(TraceType::CPU)},
{"Memory", static_cast<UInt8>(TraceType::Memory)},
};
Block TraceLogElement::createBlock()

View File

@ -14,13 +14,12 @@ struct TraceLogElement
using TraceDataType = DataTypeEnum8;
static const TraceDataType::Values trace_values;
time_t event_time;
TraceType trace_type;
UInt64 thread_id;
String query_id;
Array trace;
UInt64 size; /// Allocation size in bytes for |TraceType::MEMORY|
time_t event_time{};
TraceType trace_type{};
UInt64 thread_id{};
String query_id{};
Array trace{};
UInt64 size{}; /// Allocation size in bytes for TraceType::Memory
static std::string name() { return "TraceLog"; }
static Block createBlock();

View File

@ -4,7 +4,7 @@
#include <DataTypes/NestedUtils.h>
#include <DataTypes/DataTypeArray.h>
#include <Columns/ColumnArray.h>
#include <Interpreters/evaluateMissingDefaults.h>
#include <Interpreters/inplaceBlockConversions.h>
#include <Core/Block.h>
#include <Storages/ColumnDefault.h>

View File

@ -591,8 +591,7 @@ void executeQuery(
WriteBuffer & ostr,
bool allow_into_outfile,
Context & context,
std::function<void(const String &, const String &)> set_content_type_and_format,
std::function<void(const String &)> set_query_id)
std::function<void(const String &, const String &, const String &, const String &)> set_result_details)
{
PODArray<char> parse_buf;
const char * begin;
@ -681,11 +680,8 @@ void executeQuery(
out->onProgress(progress);
});
if (set_content_type_and_format)
set_content_type_and_format(out->getContentType(), format_name);
if (set_query_id)
set_query_id(context.getClientInfo().current_query_id);
if (set_result_details)
set_result_details(context.getClientInfo().current_query_id, out->getContentType(), format_name, DateLUT::instance().getTimeZone());
if (ast->as<ASTWatchQuery>())
{
@ -743,11 +739,8 @@ void executeQuery(
out->onProgress(progress);
});
if (set_content_type_and_format)
set_content_type_and_format(out->getContentType(), format_name);
if (set_query_id)
set_query_id(context.getClientInfo().current_query_id);
if (set_result_details)
set_result_details(context.getClientInfo().current_query_id, out->getContentType(), format_name, DateLUT::instance().getTimeZone());
pipeline.setOutput(std::move(out));

View File

@ -19,8 +19,7 @@ void executeQuery(
WriteBuffer & ostr, /// Where to write query output to.
bool allow_into_outfile, /// If true and the query contains INTO OUTFILE section, redirect output to that file.
Context & context, /// DB, tables, data types, storage engines, functions, aggregate functions...
std::function<void(const String &, const String &)> set_content_type_and_format, /// If non-empty callback is passed, it will be called with the Content-Type and the Format of the result.
std::function<void(const String &)> set_query_id /// If non-empty callback is passed, it will be called with the query id.
std::function<void(const String &, const String &, const String &, const String &)> set_result_details /// If a non-empty callback is passed, it will be called with the query id, the content-type, the format, and the timezone.
);

View File

@ -114,10 +114,10 @@ NamesAndTypesList getColumnsFromTableExpression(const ASTTableExpression & table
return getColumnsFromTableExpression(table_expression, context, materialized, aliases, virtuals);
}
std::vector<TableWithColumnNames> getDatabaseAndTablesWithColumnNames(const std::vector<const ASTTableExpression *> & table_expressions,
const Context & context, bool remove_duplicates)
std::vector<TableWithColumnNamesAndTypes> getDatabaseAndTablesWithColumns(const std::vector<const ASTTableExpression *> & table_expressions,
const Context & context)
{
std::vector<TableWithColumnNames> tables_with_columns;
std::vector<TableWithColumnNamesAndTypes> tables_with_columns;
if (!table_expressions.empty())
{
@ -125,17 +125,16 @@ std::vector<TableWithColumnNames> getDatabaseAndTablesWithColumnNames(const std:
for (const ASTTableExpression * table_expression : table_expressions)
{
DatabaseAndTableWithAlias table_name(*table_expression, current_database);
NamesAndTypesList materialized;
NamesAndTypesList aliases;
NamesAndTypesList virtuals;
NamesAndTypesList names_and_types = getColumnsFromTableExpression(*table_expression, context, materialized, aliases, virtuals);
if (remove_duplicates)
removeDuplicateColumns(names_and_types);
removeDuplicateColumns(names_and_types);
tables_with_columns.emplace_back(
DatabaseAndTableWithAlias(*table_expression, current_database), names_and_types);
tables_with_columns.emplace_back(std::move(table_name), names_and_types.getNames());
auto & table = tables_with_columns.back();
table.addHiddenColumns(materialized);
table.addHiddenColumns(aliases);
@ -146,4 +145,15 @@ std::vector<TableWithColumnNames> getDatabaseAndTablesWithColumnNames(const std:
return tables_with_columns;
}
std::vector<TableWithColumnNames> getDatabaseAndTablesWithColumnNames(const std::vector<const ASTTableExpression *> & table_expressions,
const Context & context)
{
std::vector<TableWithColumnNamesAndTypes> tables_with_columns = getDatabaseAndTablesWithColumns(table_expressions, context);
std::vector<TableWithColumnNames> out;
out.reserve(tables_with_columns.size());
for (auto & table : tables_with_columns)
out.emplace_back(table.removeTypes());
return out;
}
}

Some files were not shown because too many files have changed in this diff Show More