2018-02-01 17:55:08 +00:00
|
|
|
#pragma once
|
2018-08-30 18:48:41 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
#include <Core/SettingsEnums.h>
|
|
|
|
#include <Interpreters/Context_fwd.h>
|
|
|
|
#include <IO/Progress.h>
|
2018-02-01 17:55:08 +00:00
|
|
|
#include <Common/MemoryTracker.h>
|
2021-04-10 23:33:54 +00:00
|
|
|
#include <Common/ProfileEvents.h>
|
2021-10-02 07:13:14 +00:00
|
|
|
#include <base/StringRef.h>
|
2018-08-30 18:48:41 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
#include <boost/noncopyable.hpp>
|
2018-08-30 18:48:41 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
#include <functional>
|
2018-06-19 20:30:35 +00:00
|
|
|
#include <map>
|
2021-04-10 23:33:54 +00:00
|
|
|
#include <memory>
|
2018-03-01 16:52:24 +00:00
|
|
|
#include <mutex>
|
2021-08-30 15:35:25 +00:00
|
|
|
#include <unordered_set>
|
2018-03-01 16:52:24 +00:00
|
|
|
|
2018-02-01 17:55:08 +00:00
|
|
|
|
|
|
|
namespace Poco
|
|
|
|
{
|
|
|
|
class Logger;
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2023-01-21 10:24:57 +00:00
|
|
|
template <class T>
|
|
|
|
class ConcurrentBoundedQueue;
|
|
|
|
|
2018-02-01 17:55:08 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2018-05-17 16:01:41 +00:00
|
|
|
class QueryStatus;
|
|
|
|
class ThreadStatus;
|
2019-03-04 13:03:32 +00:00
|
|
|
class QueryProfilerReal;
|
2021-08-25 00:58:49 +00:00
|
|
|
class QueryProfilerCPU;
|
2018-05-31 15:54:08 +00:00
|
|
|
class QueryThreadLog;
|
2020-05-01 18:47:41 +00:00
|
|
|
class TasksStatsCounters;
|
2018-08-14 20:29:42 +00:00
|
|
|
struct RUsageCounters;
|
2020-02-19 16:35:01 +00:00
|
|
|
struct PerfEventsCounters;
|
2018-06-04 14:16:27 +00:00
|
|
|
class TaskStatsInfoGetter;
|
2018-06-15 17:32:35 +00:00
|
|
|
class InternalTextLogsQueue;
|
2021-08-09 17:38:29 +00:00
|
|
|
struct ViewRuntimeData;
|
2021-06-18 13:44:08 +00:00
|
|
|
class QueryViewsLog;
|
2021-10-13 20:47:28 +00:00
|
|
|
class MemoryTrackerThreadSwitcher;
|
2018-06-15 17:32:35 +00:00
|
|
|
using InternalTextLogsQueuePtr = std::shared_ptr<InternalTextLogsQueue>;
|
|
|
|
using InternalTextLogsQueueWeakPtr = std::weak_ptr<InternalTextLogsQueue>;
|
2021-09-02 14:27:19 +00:00
|
|
|
|
|
|
|
using InternalProfileEventsQueue = ConcurrentBoundedQueue<Block>;
|
|
|
|
using InternalProfileEventsQueuePtr = std::shared_ptr<InternalProfileEventsQueue>;
|
|
|
|
using InternalProfileEventsQueueWeakPtr = std::weak_ptr<InternalProfileEventsQueue>;
|
2021-08-30 15:35:25 +00:00
|
|
|
using ThreadStatusPtr = ThreadStatus *;
|
2018-02-01 17:55:08 +00:00
|
|
|
|
2018-09-06 00:28:15 +00:00
|
|
|
/** Thread group is a collection of threads dedicated to single task
|
|
|
|
* (query or other process like background merge).
|
|
|
|
*
|
|
|
|
* ProfileEvents (counters) from a thread are propagated to thread group.
|
|
|
|
*
|
|
|
|
* Create via CurrentThread::initializeQuery (for queries) or directly (for various background tasks).
|
|
|
|
* Use via CurrentThread::getGroup.
|
|
|
|
*/
|
2023-03-14 11:36:26 +00:00
|
|
|
class ThreadGroupStatus;
|
|
|
|
using ThreadGroupStatusPtr = std::shared_ptr<ThreadGroupStatus>;
|
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
class ThreadGroupStatus
|
|
|
|
{
|
|
|
|
public:
|
2023-03-15 21:12:29 +00:00
|
|
|
ThreadGroupStatus();
|
|
|
|
using FatalErrorCallback = std::function<void()>;
|
|
|
|
ThreadGroupStatus(ContextPtr query_context_, FatalErrorCallback fatal_error_callback_ = {});
|
|
|
|
|
2023-03-14 11:36:26 +00:00
|
|
|
/// The first thread created this thread group
|
2023-03-15 21:12:29 +00:00
|
|
|
const UInt64 master_thread_id;
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
/// Set up at creation, no race when reading
|
|
|
|
const ContextWeakPtr query_context;
|
|
|
|
const ContextWeakPtr global_context;
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
const FatalErrorCallback fatal_error_callback;
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
ProfileEvents::Counters performance_counters{VariableContext::Process};
|
|
|
|
MemoryTracker memory_tracker{VariableContext::Process};
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
struct SharedData
|
|
|
|
{
|
|
|
|
InternalProfileEventsQueueWeakPtr profile_queue_ptr;
|
2019-07-09 10:39:05 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
InternalTextLogsQueueWeakPtr logs_queue_ptr;
|
|
|
|
LogsLevel client_logs_level = LogsLevel::none;
|
2021-12-14 07:25:30 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
String query_for_logs;
|
|
|
|
UInt64 normalized_query_hash = 0;
|
|
|
|
};
|
2021-12-14 07:25:30 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
SharedData getSharedData()
|
|
|
|
{
|
2023-03-17 10:59:44 +00:00
|
|
|
/// Critical section for making the copy of shared_data
|
2023-03-15 21:12:29 +00:00
|
|
|
std::lock_guard lock(mutex);
|
|
|
|
return shared_data;
|
|
|
|
}
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
/// Mutation shared data
|
|
|
|
void attachInternalTextLogsQueue(const InternalTextLogsQueuePtr & logs_queue, LogsLevel logs_level);
|
|
|
|
void attachQueryForLog(const String & query_, UInt64 normalized_hash = 0);
|
|
|
|
void attachInternalProfileEventsQueue(const InternalProfileEventsQueuePtr & profile_queue);
|
2023-03-14 11:36:26 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
/// When new query starts, new thread group is created for it, current thread becomes master thread of the query
|
|
|
|
static ThreadGroupStatusPtr createForQuery(ContextPtr query_context_, FatalErrorCallback fatal_error_callback_ = {});
|
2023-03-14 11:36:26 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
std::vector<UInt64> getInvolvedThreadIds() const;
|
|
|
|
void linkThread(UInt64 thread_it);
|
2023-03-14 11:36:26 +00:00
|
|
|
|
|
|
|
private:
|
2023-03-15 21:12:29 +00:00
|
|
|
mutable std::mutex mutex;
|
2023-03-14 11:36:26 +00:00
|
|
|
|
|
|
|
/// Set up at creation, no race when reading
|
2023-03-15 21:12:29 +00:00
|
|
|
SharedData shared_data;
|
2023-03-14 11:36:26 +00:00
|
|
|
/// Set of all thread ids which has been attached to the group
|
|
|
|
std::unordered_set<UInt64> thread_ids;
|
|
|
|
};
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2022-08-11 11:27:53 +00:00
|
|
|
/**
|
|
|
|
* We use **constinit** here to tell the compiler the current_thread variable is initialized.
|
|
|
|
* If we didn't help the compiler, then it would most likely add a check before every use of the variable to initialize it if needed.
|
|
|
|
* Instead it will trust that we are doing the right thing (and we do initialize it to nullptr) and emit more optimal code.
|
|
|
|
* This is noticeable in functions like CurrentMemoryTracker::free and CurrentMemoryTracker::allocImpl
|
|
|
|
* See also:
|
|
|
|
* - https://en.cppreference.com/w/cpp/language/constinit
|
|
|
|
* - https://github.com/ClickHouse/ClickHouse/pull/40078
|
|
|
|
*/
|
2022-08-10 20:05:09 +00:00
|
|
|
extern thread_local constinit ThreadStatus * current_thread;
|
2019-01-13 18:51:57 +00:00
|
|
|
|
2018-09-06 00:28:15 +00:00
|
|
|
/** Encapsulates all per-thread info (ProfileEvents, MemoryTracker, query_id, query context, etc.).
|
2019-01-13 18:51:57 +00:00
|
|
|
* The object must be created in thread function and destroyed in the same thread before the exit.
|
|
|
|
* It is accessed through thread-local pointer.
|
2018-09-06 00:28:15 +00:00
|
|
|
*
|
|
|
|
* This object should be used only via "CurrentThread", see CurrentThread.h
|
|
|
|
*/
|
2019-01-13 18:51:57 +00:00
|
|
|
class ThreadStatus : public boost::noncopyable
|
2018-02-01 17:55:08 +00:00
|
|
|
{
|
2018-05-17 16:01:41 +00:00
|
|
|
public:
|
2018-06-04 14:16:27 +00:00
|
|
|
/// Linux's PID (or TGID) (the same id is shown by ps util)
|
2020-05-01 18:47:41 +00:00
|
|
|
const UInt64 thread_id = 0;
|
2019-07-06 18:02:28 +00:00
|
|
|
/// Also called "nice" value. If it was changed to non-zero (when attaching query) - will be reset to zero when query is detached.
|
|
|
|
Int32 os_thread_priority = 0;
|
2018-06-04 14:16:27 +00:00
|
|
|
|
2018-06-09 15:29:08 +00:00
|
|
|
/// TODO: merge them into common entity
|
|
|
|
ProfileEvents::Counters performance_counters{VariableContext::Thread};
|
2023-01-31 12:34:07 +00:00
|
|
|
/// Points to performance_counters by default.
|
2023-01-31 14:01:12 +00:00
|
|
|
/// Could be changed to point to another object to calculate performance counters for some narrow scope.
|
2023-01-31 12:34:07 +00:00
|
|
|
ProfileEvents::Counters * current_performance_counters{&performance_counters};
|
2020-04-30 13:25:17 +00:00
|
|
|
|
2023-03-14 11:36:26 +00:00
|
|
|
MemoryTracker memory_tracker{VariableContext::Thread};
|
2019-07-10 18:12:50 +00:00
|
|
|
/// Small amount of untracked memory (per thread atomic-less counter)
|
|
|
|
Int64 untracked_memory = 0;
|
2020-04-30 13:25:17 +00:00
|
|
|
/// Each thread could new/delete memory in range of (-untracked_memory_limit, untracked_memory_limit) without access to common counters.
|
|
|
|
Int64 untracked_memory_limit = 4 * 1024 * 1024;
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2018-06-01 19:39:32 +00:00
|
|
|
/// Statistics of read and write rows/bytes
|
2018-05-31 15:54:08 +00:00
|
|
|
Progress progress_in;
|
|
|
|
Progress progress_out;
|
2018-02-01 17:55:08 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
private:
|
2023-01-23 12:45:28 +00:00
|
|
|
/// Group of threads, to which this thread attached
|
2020-11-09 15:07:38 +00:00
|
|
|
ThreadGroupStatusPtr thread_group;
|
|
|
|
|
|
|
|
/// Is set once
|
2021-04-10 23:33:54 +00:00
|
|
|
ContextWeakPtr global_context;
|
2020-11-09 15:07:38 +00:00
|
|
|
/// Use it only from current thread
|
2021-04-10 23:33:54 +00:00
|
|
|
ContextWeakPtr query_context;
|
2020-11-09 15:07:38 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
/// Is used to send logs from logs_queue to client in case of fatal errors.
|
|
|
|
using FatalErrorCallback = std::function<void()>;
|
|
|
|
FatalErrorCallback fatal_error_callback;
|
|
|
|
|
2023-03-17 10:59:44 +00:00
|
|
|
ThreadGroupStatus::SharedData local_data;
|
2020-11-09 15:07:38 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
bool performance_counters_finalized = false;
|
2020-11-09 15:07:38 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
String query_id_from_query_context;
|
|
|
|
/// Requires access to query_id.
|
|
|
|
friend class MemoryTrackerThreadSwitcher;
|
|
|
|
void setQueryId(const String & query_id_)
|
|
|
|
{
|
|
|
|
query_id_from_query_context = query_id_;
|
|
|
|
}
|
2021-09-02 14:27:19 +00:00
|
|
|
|
2023-03-14 11:36:26 +00:00
|
|
|
struct TimePoint
|
|
|
|
{
|
|
|
|
void setUp();
|
2023-03-15 21:12:29 +00:00
|
|
|
UInt64 nanoseconds() const;
|
|
|
|
UInt64 microseconds() const;
|
|
|
|
UInt64 seconds() const;
|
2023-03-14 11:36:26 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
std::chrono::time_point<std::chrono::system_clock> point;
|
2023-03-14 11:36:26 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
TimePoint query_start_time{};
|
2020-11-09 15:07:38 +00:00
|
|
|
|
|
|
|
// CPU and Real time query profilers
|
|
|
|
std::unique_ptr<QueryProfilerReal> query_profiler_real;
|
2021-08-25 00:58:49 +00:00
|
|
|
std::unique_ptr<QueryProfilerCPU> query_profiler_cpu;
|
2020-11-09 15:07:38 +00:00
|
|
|
|
|
|
|
/// Use ptr not to add extra dependencies in the header
|
|
|
|
std::unique_ptr<RUsageCounters> last_rusage;
|
|
|
|
std::unique_ptr<TasksStatsCounters> taskstats;
|
|
|
|
|
2022-10-31 11:44:28 +00:00
|
|
|
/// See setInternalThread()
|
|
|
|
bool internal_thread = false;
|
2021-07-01 11:27:11 +00:00
|
|
|
|
2023-03-14 11:36:26 +00:00
|
|
|
/// This is helpful for cut linking dependencies for clickhouse_common_io
|
|
|
|
using Deleter = std::function<void()>;
|
|
|
|
Deleter deleter;
|
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
Poco::Logger * log = nullptr;
|
|
|
|
|
2020-11-09 15:07:38 +00:00
|
|
|
public:
|
|
|
|
ThreadStatus();
|
|
|
|
~ThreadStatus();
|
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
ThreadGroupStatusPtr getThreadGroup() const;
|
2018-02-01 17:55:08 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
const String & getQueryId() const;
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
ContextPtr getQueryContext() const;
|
|
|
|
ContextPtr getGlobalContext() const;
|
2022-03-29 17:49:42 +00:00
|
|
|
|
2022-10-31 11:44:28 +00:00
|
|
|
/// "Internal" ThreadStatus is used for materialized views for separate
|
|
|
|
/// tracking into system.query_views_log
|
|
|
|
///
|
|
|
|
/// You can have multiple internal threads, but only one non-internal with
|
|
|
|
/// the same thread_id.
|
|
|
|
///
|
|
|
|
/// "Internal" thread:
|
|
|
|
/// - cannot have query profiler
|
|
|
|
/// since the running (main query) thread should already have one
|
|
|
|
/// - should not try to obtain latest counter on detach
|
|
|
|
/// because detaching of such threads will be done from a different
|
|
|
|
/// thread_id, and some counters are not available (i.e. getrusage()),
|
|
|
|
/// but anyway they are accounted correctly in the main ThreadStatus of a
|
|
|
|
/// query.
|
|
|
|
void setInternalThread();
|
2021-08-13 16:32:29 +00:00
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
/// Attaches slave thread to existing thread group
|
2023-03-15 21:12:29 +00:00
|
|
|
void attachToGroup(const ThreadGroupStatusPtr & thread_group_, bool check_detached = true);
|
2023-03-14 11:36:26 +00:00
|
|
|
|
|
|
|
/// Detaches thread from the thread group and the query, dumps performance counters if they have not been dumped
|
2023-03-15 21:12:29 +00:00
|
|
|
void detachFromGroup();
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2023-02-08 11:12:59 +00:00
|
|
|
/// Returns pointer to the current profile counters to restore them back.
|
|
|
|
/// Note: consequent call with new scope will detach previous scope.
|
|
|
|
ProfileEvents::Counters * attachProfileCountersScope(ProfileEvents::Counters * performance_counters_scope);
|
2023-01-23 12:45:28 +00:00
|
|
|
|
2019-07-10 12:19:17 +00:00
|
|
|
void attachInternalTextLogsQueue(const InternalTextLogsQueuePtr & logs_queue,
|
|
|
|
LogsLevel client_logs_level);
|
2023-03-15 21:12:29 +00:00
|
|
|
InternalTextLogsQueuePtr getInternalTextLogsQueue() const;
|
|
|
|
LogsLevel getClientLogsLevel() const;
|
2021-09-02 14:27:19 +00:00
|
|
|
|
|
|
|
void attachInternalProfileEventsQueue(const InternalProfileEventsQueuePtr & profile_queue);
|
2023-03-15 21:12:29 +00:00
|
|
|
InternalProfileEventsQueuePtr getInternalProfileEventsQueue() const;
|
|
|
|
|
|
|
|
void attachQueryForLog(const String & query_);
|
|
|
|
const String & getQueryForLog() const;
|
2021-09-02 14:27:19 +00:00
|
|
|
|
2023-03-14 11:36:26 +00:00
|
|
|
/// Proper cal for fatal_error_callback
|
2020-06-20 11:17:15 +00:00
|
|
|
void onFatalError();
|
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
/// Update several ProfileEvents counters
|
|
|
|
void updatePerformanceCounters();
|
|
|
|
|
|
|
|
/// Update ProfileEvents and dumps info to system.query_thread_log
|
|
|
|
void finalizePerformanceCounters();
|
|
|
|
|
2021-08-09 17:38:29 +00:00
|
|
|
/// Set the counters last usage to now
|
|
|
|
void resetPerformanceCountersLastUsage();
|
|
|
|
|
2021-08-26 08:01:26 +00:00
|
|
|
void logToQueryViewsLog(const ViewRuntimeData & vinfo);
|
2021-06-18 13:44:08 +00:00
|
|
|
|
2023-03-01 17:16:03 +00:00
|
|
|
void flushUntrackedMemory();
|
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
private:
|
2020-10-16 19:33:19 +00:00
|
|
|
void applyQuerySettings();
|
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
void initPerformanceCounters();
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2018-12-25 00:19:17 +00:00
|
|
|
void initQueryProfiler();
|
|
|
|
|
|
|
|
void finalizeQueryProfiler();
|
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
void logToQueryThreadLog(QueryThreadLog & thread_log, const String & current_database);
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2023-03-15 21:12:29 +00:00
|
|
|
void attachToGroupImpl(const ThreadGroupStatusPtr & thread_group_);
|
2018-02-01 17:55:08 +00:00
|
|
|
};
|
|
|
|
|
Use total_memory_tracker when there is no other MemoryTracker object.
This should significantly reduce the MemoryTracking drift, test shows
that there is 0 drift after query storm (100 queries, via http/tcp/tcp
in one session).
TL;DR;
To track memory, clickhouse creates memory tracker object for each
thread **explicitly**, but until it is not created the memory
allocations are not under account.
There should not be lot of allocations w/o memory tracker, since most of
the time it is created early enough, but even this maybe enough to
trigger some problems.
Plus sometimes it is not possible to create it, for example some 3d
party library does not allow to do this explicitly:
- for example before #15740 allocations from librdkafka threads,
- or even worse, poco threads, they don't have any routines to do this.
This won't be a problem for `MemoryTracking` metric if the deallocation
will be done from the same thread w/o memory tracker (or vise versa),
but this is not always true.
NOTE, that this will slow down per-thread allocations w/o memory
tracker, since before this patch there were no memory tracking for them
while now they will be accounted in total_memory_tracker, and for
total_memory_tracker max_untracked_memory is always reached.
But this should not be significant.
2020-10-18 07:32:49 +00:00
|
|
|
/**
|
|
|
|
* Creates ThreadStatus for the main thread.
|
|
|
|
*/
|
|
|
|
class MainThreadStatus : public ThreadStatus
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
static MainThreadStatus & getInstance();
|
|
|
|
static ThreadStatus * get() { return main_thread; }
|
|
|
|
static bool isMainThread() { return main_thread == current_thread; }
|
|
|
|
|
|
|
|
~MainThreadStatus();
|
|
|
|
|
|
|
|
private:
|
|
|
|
MainThreadStatus();
|
|
|
|
|
|
|
|
static ThreadStatus * main_thread;
|
|
|
|
};
|
|
|
|
|
2018-02-01 17:55:08 +00:00
|
|
|
}
|