2018-02-01 17:55:08 +00:00
|
|
|
#pragma once
|
2018-08-30 18:48:41 +00:00
|
|
|
|
2019-07-05 14:15:05 +00:00
|
|
|
#include <common/StringRef.h>
|
2018-02-01 17:55:08 +00:00
|
|
|
#include <Common/ProfileEvents.h>
|
|
|
|
#include <Common/MemoryTracker.h>
|
2018-08-30 18:48:41 +00:00
|
|
|
|
2020-07-16 22:01:08 +00:00
|
|
|
#include <Core/SettingsEnums.h>
|
2019-07-09 10:39:05 +00:00
|
|
|
|
2018-05-31 15:54:08 +00:00
|
|
|
#include <IO/Progress.h>
|
2018-08-30 18:48:41 +00:00
|
|
|
|
2018-02-01 17:55:08 +00:00
|
|
|
#include <memory>
|
2018-06-19 20:30:35 +00:00
|
|
|
#include <map>
|
2018-03-01 16:52:24 +00:00
|
|
|
#include <mutex>
|
2018-06-19 20:30:35 +00:00
|
|
|
#include <shared_mutex>
|
2019-01-13 18:51:57 +00:00
|
|
|
#include <functional>
|
|
|
|
#include <boost/noncopyable.hpp>
|
2018-03-01 16:52:24 +00:00
|
|
|
|
2018-02-01 17:55:08 +00:00
|
|
|
|
|
|
|
namespace Poco
|
|
|
|
{
|
|
|
|
class Logger;
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2018-05-31 15:54:08 +00:00
|
|
|
class Context;
|
2018-05-17 16:01:41 +00:00
|
|
|
class QueryStatus;
|
|
|
|
class ThreadStatus;
|
2019-03-04 13:03:32 +00:00
|
|
|
class QueryProfilerReal;
|
|
|
|
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;
|
|
|
|
using InternalTextLogsQueuePtr = std::shared_ptr<InternalTextLogsQueue>;
|
|
|
|
using InternalTextLogsQueueWeakPtr = std::weak_ptr<InternalTextLogsQueue>;
|
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.
|
|
|
|
*/
|
2018-06-19 20:30:35 +00:00
|
|
|
class ThreadGroupStatus
|
|
|
|
{
|
|
|
|
public:
|
2019-02-08 13:23:10 +00:00
|
|
|
mutable std::mutex mutex;
|
2018-06-19 20:30:35 +00:00
|
|
|
|
|
|
|
ProfileEvents::Counters performance_counters{VariableContext::Process};
|
|
|
|
MemoryTracker memory_tracker{VariableContext::Process};
|
|
|
|
|
|
|
|
Context * query_context = nullptr;
|
|
|
|
Context * global_context = nullptr;
|
|
|
|
|
|
|
|
InternalTextLogsQueueWeakPtr logs_queue_ptr;
|
2020-07-09 04:15:45 +00:00
|
|
|
std::function<void()> fatal_error_callback;
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2020-02-02 20:01:13 +00:00
|
|
|
std::vector<UInt64> thread_ids;
|
2018-06-20 15:21:42 +00:00
|
|
|
|
|
|
|
/// The first thread created this thread group
|
2020-02-02 20:01:13 +00:00
|
|
|
UInt64 master_thread_id = 0;
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2019-07-16 16:27:42 +00:00
|
|
|
LogsLevel client_logs_level = LogsLevel::none;
|
2019-07-09 10:39:05 +00:00
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
String query;
|
|
|
|
};
|
|
|
|
|
|
|
|
using ThreadGroupStatusPtr = std::shared_ptr<ThreadGroupStatus>;
|
|
|
|
|
|
|
|
|
2019-02-08 13:23:10 +00:00
|
|
|
extern thread_local 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:
|
2019-01-13 18:51:57 +00:00
|
|
|
ThreadStatus();
|
|
|
|
~ThreadStatus();
|
|
|
|
|
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};
|
|
|
|
MemoryTracker memory_tracker{VariableContext::Thread};
|
2020-04-30 13:25:17 +00:00
|
|
|
|
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
|
|
|
|
2019-01-13 18:51:57 +00:00
|
|
|
using Deleter = std::function<void()>;
|
|
|
|
Deleter deleter;
|
2018-02-01 17:55:08 +00:00
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
ThreadGroupStatusPtr getThreadGroup() const
|
|
|
|
{
|
|
|
|
return thread_group;
|
|
|
|
}
|
2018-02-01 17:55:08 +00:00
|
|
|
|
2018-06-01 19:39:32 +00:00
|
|
|
enum ThreadState
|
2018-05-31 15:54:08 +00:00
|
|
|
{
|
2018-06-19 20:30:35 +00:00
|
|
|
DetachedFromQuery = 0, /// We just created thread or it is a background thread
|
2018-06-01 19:39:32 +00:00
|
|
|
AttachedToQuery, /// Thread executes enqueued query
|
|
|
|
Died, /// Thread does not exist
|
|
|
|
};
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2018-06-01 19:39:32 +00:00
|
|
|
int getCurrentState() const
|
2018-05-31 15:54:08 +00:00
|
|
|
{
|
2018-06-01 19:39:32 +00:00
|
|
|
return thread_state.load(std::memory_order_relaxed);
|
2018-05-31 15:54:08 +00:00
|
|
|
}
|
|
|
|
|
2019-07-10 20:47:39 +00:00
|
|
|
StringRef getQueryId() const
|
|
|
|
{
|
|
|
|
return query_id;
|
|
|
|
}
|
2018-06-19 20:30:35 +00:00
|
|
|
|
2018-07-04 17:28:06 +00:00
|
|
|
/// Starts new query and create new thread group for it, current thread becomes master thread of the query
|
2018-06-19 20:30:35 +00:00
|
|
|
void initializeQuery();
|
|
|
|
|
|
|
|
/// Attaches slave thread to existing thread group
|
|
|
|
void attachQuery(const ThreadGroupStatusPtr & thread_group_, bool check_detached = true);
|
|
|
|
|
2018-06-15 17:32:35 +00:00
|
|
|
InternalTextLogsQueuePtr getInternalTextLogsQueue() const
|
2018-06-06 20:57:07 +00:00
|
|
|
{
|
|
|
|
return thread_state == Died ? nullptr : logs_queue_ptr.lock();
|
|
|
|
}
|
|
|
|
|
2019-07-10 12:19:17 +00:00
|
|
|
void attachInternalTextLogsQueue(const InternalTextLogsQueuePtr & logs_queue,
|
|
|
|
LogsLevel client_logs_level);
|
2018-06-06 20:57:07 +00:00
|
|
|
|
2020-06-20 11:17:15 +00:00
|
|
|
/// Callback that is used to trigger sending fatal error messages to client.
|
|
|
|
void setFatalErrorCallback(std::function<void()> callback);
|
|
|
|
void onFatalError();
|
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
/// Sets query context for current thread and its thread group
|
|
|
|
/// NOTE: query_context have to be alive until detachQuery() is called
|
|
|
|
void attachQueryContext(Context & query_context);
|
|
|
|
|
|
|
|
/// Update several ProfileEvents counters
|
|
|
|
void updatePerformanceCounters();
|
|
|
|
|
|
|
|
/// Update ProfileEvents and dumps info to system.query_thread_log
|
|
|
|
void finalizePerformanceCounters();
|
|
|
|
|
|
|
|
/// Detaches thread from the thread group and the query, dumps performance counters if they have not been dumped
|
|
|
|
void detachQuery(bool exit_if_already_detached = false, bool thread_exits = false);
|
2018-06-09 15:29:08 +00:00
|
|
|
|
2018-05-29 18:14:31 +00:00
|
|
|
protected:
|
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();
|
|
|
|
|
2020-10-29 19:28:46 +00:00
|
|
|
void logToQueryThreadLog(QueryThreadLog & thread_log, const String & current_database, std::chrono::time_point<std::chrono::system_clock> now);
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2020-04-22 05:39:31 +00:00
|
|
|
void assertState(const std::initializer_list<int> & permitted_states, const char * description = nullptr) const;
|
2018-05-29 18:14:31 +00:00
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
ThreadGroupStatusPtr thread_group;
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
std::atomic<int> thread_state{ThreadState::DetachedFromQuery};
|
2018-05-31 15:54:08 +00:00
|
|
|
|
|
|
|
/// Is set once
|
2018-06-19 20:30:35 +00:00
|
|
|
Context * global_context = nullptr;
|
2018-06-01 19:39:32 +00:00
|
|
|
/// Use it only from current thread
|
|
|
|
Context * query_context = nullptr;
|
|
|
|
|
2019-02-04 15:39:08 +00:00
|
|
|
String query_id;
|
|
|
|
|
2018-06-06 20:57:07 +00:00
|
|
|
/// A logs queue used by TCPHandler to pass logs to a client
|
2018-06-15 17:32:35 +00:00
|
|
|
InternalTextLogsQueueWeakPtr logs_queue_ptr;
|
2018-06-06 20:57:07 +00:00
|
|
|
|
2018-06-19 20:30:35 +00:00
|
|
|
bool performance_counters_finalized = false;
|
2018-06-01 19:39:32 +00:00
|
|
|
UInt64 query_start_time_nanoseconds = 0;
|
2020-08-18 21:41:01 +00:00
|
|
|
UInt64 query_start_time_microseconds = 0;
|
2018-06-01 19:39:32 +00:00
|
|
|
time_t query_start_time = 0;
|
2018-06-04 14:16:27 +00:00
|
|
|
size_t queries_started = 0;
|
2018-05-31 15:54:08 +00:00
|
|
|
|
2019-03-04 13:03:32 +00:00
|
|
|
// CPU and Real time query profilers
|
|
|
|
std::unique_ptr<QueryProfilerReal> query_profiler_real;
|
|
|
|
std::unique_ptr<QueryProfilerCpu> query_profiler_cpu;
|
2018-12-25 00:19:17 +00:00
|
|
|
|
2018-05-31 15:54:08 +00:00
|
|
|
Poco::Logger * log = nullptr;
|
2018-02-01 17:55:08 +00:00
|
|
|
|
2018-05-29 18:14:31 +00:00
|
|
|
friend class CurrentThread;
|
|
|
|
|
2018-06-06 20:57:07 +00:00
|
|
|
/// Use ptr not to add extra dependencies in the header
|
2018-08-14 20:29:42 +00:00
|
|
|
std::unique_ptr<RUsageCounters> last_rusage;
|
2020-05-01 18:47:41 +00:00
|
|
|
std::unique_ptr<TasksStatsCounters> taskstats;
|
2018-08-22 00:41:30 +00:00
|
|
|
|
2020-06-20 11:17:15 +00:00
|
|
|
/// Is used to send logs from logs_queue to client in case of fatal errors.
|
|
|
|
std::function<void()> fatal_error_callback;
|
|
|
|
|
2020-01-28 12:01:45 +00:00
|
|
|
private:
|
|
|
|
void setupState(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
|
|
|
}
|