2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/AsynchronousMetrics.h>
|
2020-06-10 19:17:30 +00:00
|
|
|
#include <Interpreters/AsynchronousMetricLog.h>
|
2018-09-03 10:14:05 +00:00
|
|
|
#include <Interpreters/ExpressionJIT.h>
|
2020-02-10 13:10:17 +00:00
|
|
|
#include <Interpreters/DatabaseCatalog.h>
|
2020-05-20 20:16:32 +00:00
|
|
|
#include <Interpreters/Context.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/Exception.h>
|
|
|
|
#include <Common/setThreadName.h>
|
|
|
|
#include <Common/CurrentMetrics.h>
|
2017-07-13 20:58:19 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
2020-12-17 13:47:03 +00:00
|
|
|
#include <Server/ProtocolServerAdapter.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/MarkCache.h>
|
|
|
|
#include <Storages/StorageMergeTree.h>
|
|
|
|
#include <Storages/StorageReplicatedMergeTree.h>
|
|
|
|
#include <IO/UncompressedCache.h>
|
|
|
|
#include <Databases/IDatabase.h>
|
2016-10-23 06:12:50 +00:00
|
|
|
#include <chrono>
|
|
|
|
|
2020-04-17 04:09:41 +00:00
|
|
|
|
2020-04-16 12:31:57 +00:00
|
|
|
#if !defined(ARCADIA_BUILD)
|
|
|
|
# include "config_core.h"
|
2018-06-19 18:09:09 +00:00
|
|
|
#endif
|
2017-01-20 17:58:07 +00:00
|
|
|
|
2018-08-02 00:20:20 +00:00
|
|
|
#if USE_JEMALLOC
|
2020-04-16 12:31:57 +00:00
|
|
|
# include <jemalloc/jemalloc.h>
|
2018-08-02 00:20:20 +00:00
|
|
|
#endif
|
|
|
|
|
2016-10-23 06:12:50 +00:00
|
|
|
|
2020-04-25 12:36:01 +00:00
|
|
|
namespace CurrentMetrics
|
|
|
|
{
|
|
|
|
extern const Metric MemoryTracking;
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-10-23 06:12:50 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2016-10-24 04:06:27 +00:00
|
|
|
AsynchronousMetrics::~AsynchronousMetrics()
|
2016-10-23 06:12:50 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
try
|
|
|
|
{
|
|
|
|
{
|
2020-06-10 19:17:30 +00:00
|
|
|
std::lock_guard lock{mutex};
|
2017-04-01 07:20:54 +00:00
|
|
|
quit = true;
|
|
|
|
}
|
|
|
|
|
|
|
|
wait_cond.notify_one();
|
2020-12-17 13:47:03 +00:00
|
|
|
if (thread)
|
|
|
|
thread->join();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
DB::tryLogCurrentException(__PRETTY_FUNCTION__);
|
|
|
|
}
|
2016-10-23 06:12:50 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
AsynchronousMetricValues AsynchronousMetrics::getValues() const
|
2016-10-23 06:12:50 +00:00
|
|
|
{
|
2020-06-10 19:17:30 +00:00
|
|
|
std::lock_guard lock{mutex};
|
|
|
|
return values;
|
2016-10-24 04:06:27 +00:00
|
|
|
}
|
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
static auto get_next_update_time(std::chrono::seconds update_period)
|
2016-10-24 04:06:27 +00:00
|
|
|
{
|
2020-06-26 00:16:58 +00:00
|
|
|
using namespace std::chrono;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
const auto now = time_point_cast<seconds>(system_clock::now());
|
2020-06-25 20:36:50 +00:00
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
// Use seconds since the start of the hour, because we don't know when
|
|
|
|
// the epoch started, maybe on some weird fractional time.
|
|
|
|
const auto start_of_hour = time_point_cast<seconds>(time_point_cast<hours>(now));
|
|
|
|
const auto seconds_passed = now - start_of_hour;
|
2020-06-18 01:54:10 +00:00
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
// Rotate time forward by half a period -- e.g. if a period is a minute,
|
|
|
|
// we'll collect metrics on start of minute + 30 seconds. This is to
|
|
|
|
// achieve temporal separation with MetricTransmitter. Don't forget to
|
|
|
|
// rotate it back.
|
|
|
|
const auto rotation = update_period / 2;
|
2020-06-18 01:54:10 +00:00
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
const auto periods_passed = (seconds_passed + rotation) / update_period;
|
|
|
|
const auto seconds_next = (periods_passed + 1) * update_period - rotation;
|
|
|
|
const auto time_next = start_of_hour + seconds_next;
|
2020-06-18 01:54:10 +00:00
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
return time_next;
|
|
|
|
}
|
2020-06-18 01:54:10 +00:00
|
|
|
|
2020-06-26 00:16:58 +00:00
|
|
|
void AsynchronousMetrics::run()
|
|
|
|
{
|
|
|
|
setThreadName("AsyncMetrics");
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
while (true)
|
|
|
|
{
|
2020-06-18 01:54:10 +00:00
|
|
|
{
|
|
|
|
// Wait first, so that the first metric collection is also on even time.
|
|
|
|
std::unique_lock lock{mutex};
|
2020-06-26 00:16:58 +00:00
|
|
|
if (wait_cond.wait_until(lock, get_next_update_time(update_period),
|
|
|
|
[this] { return quit; }))
|
|
|
|
{
|
2020-06-18 01:54:10 +00:00
|
|
|
break;
|
2020-06-26 00:16:58 +00:00
|
|
|
}
|
2020-06-18 01:54:10 +00:00
|
|
|
}
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
try
|
|
|
|
{
|
|
|
|
update();
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
tryLogCurrentException(__PRETTY_FUNCTION__);
|
|
|
|
}
|
|
|
|
}
|
2016-10-23 06:12:50 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-10-23 06:38:53 +00:00
|
|
|
template <typename Max, typename T>
|
|
|
|
static void calculateMax(Max & max, T x)
|
2016-10-23 06:12:50 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (Max(x) > max)
|
|
|
|
max = x;
|
2016-10-23 06:12:50 +00:00
|
|
|
}
|
|
|
|
|
2016-10-23 06:38:53 +00:00
|
|
|
template <typename Max, typename Sum, typename T>
|
|
|
|
static void calculateMaxAndSum(Max & max, Sum & sum, T x)
|
2016-10-23 06:12:50 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
sum += x;
|
|
|
|
if (Max(x) > max)
|
|
|
|
max = x;
|
2016-10-23 06:12:50 +00:00
|
|
|
}
|
|
|
|
|
2020-06-18 01:54:10 +00:00
|
|
|
#if USE_JEMALLOC && JEMALLOC_VERSION_MAJOR >= 4
|
|
|
|
uint64_t updateJemallocEpoch()
|
|
|
|
{
|
|
|
|
uint64_t value = 0;
|
|
|
|
size_t size = sizeof(value);
|
|
|
|
mallctl("epoch", &value, &size, &value, size);
|
|
|
|
return value;
|
|
|
|
}
|
|
|
|
|
|
|
|
template <typename Value>
|
|
|
|
static void saveJemallocMetricImpl(AsynchronousMetricValues & values,
|
|
|
|
const std::string & jemalloc_full_name,
|
|
|
|
const std::string & clickhouse_full_name)
|
|
|
|
{
|
|
|
|
Value value{};
|
|
|
|
size_t size = sizeof(value);
|
|
|
|
mallctl(jemalloc_full_name.c_str(), &value, &size, nullptr, 0);
|
|
|
|
values[clickhouse_full_name] = value;
|
|
|
|
}
|
|
|
|
|
|
|
|
template<typename Value>
|
|
|
|
static void saveJemallocMetric(AsynchronousMetricValues & values,
|
|
|
|
const std::string & metric_name)
|
|
|
|
{
|
|
|
|
saveJemallocMetricImpl<Value>(values,
|
|
|
|
fmt::format("stats.{}", metric_name),
|
|
|
|
fmt::format("jemalloc.{}", metric_name));
|
|
|
|
}
|
|
|
|
|
|
|
|
template<typename Value>
|
|
|
|
static void saveAllArenasMetric(AsynchronousMetricValues & values,
|
|
|
|
const std::string & metric_name)
|
|
|
|
{
|
|
|
|
saveJemallocMetricImpl<Value>(values,
|
|
|
|
fmt::format("stats.arenas.{}.{}", MALLCTL_ARENAS_ALL, metric_name),
|
|
|
|
fmt::format("jemalloc.arenas.all.{}", metric_name));
|
|
|
|
}
|
|
|
|
#endif
|
2016-10-23 06:12:50 +00:00
|
|
|
|
2016-10-24 04:06:27 +00:00
|
|
|
void AsynchronousMetrics::update()
|
2016-10-23 06:12:50 +00:00
|
|
|
{
|
2020-06-10 19:17:30 +00:00
|
|
|
AsynchronousMetricValues new_values;
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2020-12-17 13:47:03 +00:00
|
|
|
if (auto mark_cache = global_context.getMarkCache())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["MarkCacheBytes"] = mark_cache->weight();
|
|
|
|
new_values["MarkCacheFiles"] = mark_cache->count();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
{
|
2020-12-17 13:47:03 +00:00
|
|
|
if (auto uncompressed_cache = global_context.getUncompressedCache())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["UncompressedCacheBytes"] = uncompressed_cache->weight();
|
|
|
|
new_values["UncompressedCacheCells"] = uncompressed_cache->count();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-09-03 10:14:05 +00:00
|
|
|
#if USE_EMBEDDED_COMPILER
|
|
|
|
{
|
2021-03-04 17:38:12 +00:00
|
|
|
if (auto * compiled_expression_cache = CompiledExpressionCacheFactory::instance().tryGetCache())
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["CompiledExpressionCacheCount"] = compiled_expression_cache->count();
|
2018-09-03 10:14:05 +00:00
|
|
|
}
|
|
|
|
#endif
|
|
|
|
|
2020-12-17 13:47:03 +00:00
|
|
|
new_values["Uptime"] = global_context.getUptimeSeconds();
|
2017-09-07 04:02:29 +00:00
|
|
|
|
2020-04-17 04:09:41 +00:00
|
|
|
/// Process memory usage according to OS
|
|
|
|
#if defined(OS_LINUX)
|
|
|
|
{
|
2020-04-19 20:49:13 +00:00
|
|
|
MemoryStatisticsOS::Data data = memory_stat.get();
|
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["MemoryVirtual"] = data.virt;
|
|
|
|
new_values["MemoryResident"] = data.resident;
|
|
|
|
new_values["MemoryShared"] = data.shared;
|
|
|
|
new_values["MemoryCode"] = data.code;
|
|
|
|
new_values["MemoryDataAndStack"] = data.data_and_stack;
|
2020-04-19 21:43:06 +00:00
|
|
|
|
|
|
|
/// We must update the value of total_memory_tracker periodically.
|
|
|
|
/// Otherwise it might be calculated incorrectly - it can include a "drift" of memory amount.
|
|
|
|
/// See https://github.com/ClickHouse/ClickHouse/issues/10293
|
2020-10-30 19:02:02 +00:00
|
|
|
{
|
|
|
|
Int64 amount = total_memory_tracker.get();
|
|
|
|
Int64 peak = total_memory_tracker.getPeak();
|
2020-12-22 07:13:22 +00:00
|
|
|
Int64 new_amount = data.resident;
|
2020-10-30 19:02:02 +00:00
|
|
|
|
|
|
|
LOG_DEBUG(&Poco::Logger::get("AsynchronousMetrics"),
|
|
|
|
"MemoryTracking: was {}, peak {}, will set to {} (RSS), difference: {}",
|
|
|
|
ReadableSize(amount),
|
|
|
|
ReadableSize(peak),
|
2020-12-22 07:13:22 +00:00
|
|
|
ReadableSize(new_amount),
|
|
|
|
ReadableSize(new_amount - amount)
|
2020-10-30 19:02:02 +00:00
|
|
|
);
|
|
|
|
|
2020-12-22 07:13:22 +00:00
|
|
|
total_memory_tracker.set(new_amount);
|
|
|
|
CurrentMetrics::set(CurrentMetrics::MemoryTracking, new_amount);
|
2020-10-30 19:02:02 +00:00
|
|
|
}
|
2020-04-17 04:09:41 +00:00
|
|
|
}
|
|
|
|
#endif
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2020-02-10 13:10:17 +00:00
|
|
|
auto databases = DatabaseCatalog::instance().getDatabases();
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
size_t max_queue_size = 0;
|
|
|
|
size_t max_inserts_in_queue = 0;
|
|
|
|
size_t max_merges_in_queue = 0;
|
|
|
|
|
|
|
|
size_t sum_queue_size = 0;
|
|
|
|
size_t sum_inserts_in_queue = 0;
|
|
|
|
size_t sum_merges_in_queue = 0;
|
|
|
|
|
|
|
|
size_t max_absolute_delay = 0;
|
|
|
|
size_t max_relative_delay = 0;
|
|
|
|
|
|
|
|
size_t max_part_count_for_partition = 0;
|
|
|
|
|
2019-07-17 15:36:28 +00:00
|
|
|
size_t number_of_databases = databases.size();
|
|
|
|
size_t total_number_of_tables = 0;
|
|
|
|
|
2020-12-22 10:34:35 +00:00
|
|
|
size_t total_number_of_bytes = 0;
|
|
|
|
size_t total_number_of_rows = 0;
|
|
|
|
size_t total_number_of_parts = 0;
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
for (const auto & db : databases)
|
|
|
|
{
|
2020-10-15 15:57:17 +00:00
|
|
|
/// Check if database can contain MergeTree tables
|
|
|
|
if (!db.second->canContainMergeTreeTables())
|
2019-10-01 12:44:17 +00:00
|
|
|
continue;
|
2020-12-17 13:47:03 +00:00
|
|
|
for (auto iterator = db.second->getTablesIterator(global_context); iterator->isValid(); iterator->next())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2019-07-17 15:36:28 +00:00
|
|
|
++total_number_of_tables;
|
2020-04-22 06:01:33 +00:00
|
|
|
const auto & table = iterator->table();
|
2020-06-02 02:06:16 +00:00
|
|
|
if (!table)
|
|
|
|
continue;
|
|
|
|
|
2017-11-04 16:46:14 +00:00
|
|
|
StorageMergeTree * table_merge_tree = dynamic_cast<StorageMergeTree *>(table.get());
|
|
|
|
StorageReplicatedMergeTree * table_replicated_merge_tree = dynamic_cast<StorageReplicatedMergeTree *>(table.get());
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (table_replicated_merge_tree)
|
|
|
|
{
|
|
|
|
StorageReplicatedMergeTree::Status status;
|
|
|
|
table_replicated_merge_tree->getStatus(status, false);
|
|
|
|
|
|
|
|
calculateMaxAndSum(max_queue_size, sum_queue_size, status.queue.queue_size);
|
|
|
|
calculateMaxAndSum(max_inserts_in_queue, sum_inserts_in_queue, status.queue.inserts_in_queue);
|
|
|
|
calculateMaxAndSum(max_merges_in_queue, sum_merges_in_queue, status.queue.merges_in_queue);
|
|
|
|
|
2019-08-17 21:18:22 +00:00
|
|
|
if (!status.is_readonly)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2019-08-17 21:18:22 +00:00
|
|
|
try
|
|
|
|
{
|
|
|
|
time_t absolute_delay = 0;
|
|
|
|
time_t relative_delay = 0;
|
|
|
|
table_replicated_merge_tree->getReplicaDelays(absolute_delay, relative_delay);
|
|
|
|
|
|
|
|
calculateMax(max_absolute_delay, absolute_delay);
|
|
|
|
calculateMax(max_relative_delay, relative_delay);
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
tryLogCurrentException(__PRETTY_FUNCTION__,
|
|
|
|
"Cannot get replica delay for table: " + backQuoteIfNeed(db.first) + "." + backQuoteIfNeed(iterator->name()));
|
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
2019-05-03 02:00:57 +00:00
|
|
|
calculateMax(max_part_count_for_partition, table_replicated_merge_tree->getMaxPartsCountForPartition());
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (table_merge_tree)
|
|
|
|
{
|
2019-05-03 02:00:57 +00:00
|
|
|
calculateMax(max_part_count_for_partition, table_merge_tree->getMaxPartsCountForPartition());
|
2020-12-22 10:34:35 +00:00
|
|
|
const auto & settings = global_context.getSettingsRef();
|
|
|
|
total_number_of_bytes += table_merge_tree->totalBytes(settings).value();
|
|
|
|
total_number_of_rows += table_merge_tree->totalRows(settings).value();
|
|
|
|
total_number_of_parts += table_merge_tree->getPartsCount();
|
|
|
|
}
|
|
|
|
if (table_replicated_merge_tree)
|
|
|
|
{
|
|
|
|
const auto & settings = global_context.getSettingsRef();
|
|
|
|
total_number_of_bytes += table_replicated_merge_tree->totalBytes(settings).value();
|
|
|
|
total_number_of_rows += table_replicated_merge_tree->totalRows(settings).value();
|
|
|
|
total_number_of_parts += table_replicated_merge_tree->getPartsCount();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["ReplicasMaxQueueSize"] = max_queue_size;
|
|
|
|
new_values["ReplicasMaxInsertsInQueue"] = max_inserts_in_queue;
|
|
|
|
new_values["ReplicasMaxMergesInQueue"] = max_merges_in_queue;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["ReplicasSumQueueSize"] = sum_queue_size;
|
|
|
|
new_values["ReplicasSumInsertsInQueue"] = sum_inserts_in_queue;
|
|
|
|
new_values["ReplicasSumMergesInQueue"] = sum_merges_in_queue;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["ReplicasMaxAbsoluteDelay"] = max_absolute_delay;
|
|
|
|
new_values["ReplicasMaxRelativeDelay"] = max_relative_delay;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["MaxPartCountForPartition"] = max_part_count_for_partition;
|
2019-07-17 15:36:28 +00:00
|
|
|
|
2020-06-10 19:17:30 +00:00
|
|
|
new_values["NumberOfDatabases"] = number_of_databases;
|
|
|
|
new_values["NumberOfTables"] = total_number_of_tables;
|
2020-12-17 13:47:03 +00:00
|
|
|
|
2020-12-22 10:34:35 +00:00
|
|
|
new_values["TotalBytesOfMergeTreeTables"] = total_number_of_bytes;
|
|
|
|
new_values["TotalRowsOfMergeTreeTables"] = total_number_of_rows;
|
|
|
|
new_values["TotalPartsOfMergeTreeTables"] = total_number_of_parts;
|
|
|
|
|
2020-12-17 13:47:03 +00:00
|
|
|
auto get_metric_name = [](const String & name) -> const char *
|
|
|
|
{
|
|
|
|
static std::map<String, const char *> metric_map = {
|
|
|
|
{"tcp_port", "TCPThreads"},
|
|
|
|
{"tcp_port_secure", "TCPSecureThreads"},
|
|
|
|
{"http_port", "HTTPThreads"},
|
|
|
|
{"https_port", "HTTPSecureThreads"},
|
|
|
|
{"interserver_http_port", "InterserverThreads"},
|
|
|
|
{"interserver_https_port", "InterserverSecureThreads"},
|
|
|
|
{"mysql_port", "MySQLThreads"},
|
|
|
|
{"postgresql_port", "PostgreSQLThreads"},
|
|
|
|
{"grpc_port", "GRPCThreads"},
|
|
|
|
{"prometheus.port", "PrometheusThreads"}
|
|
|
|
};
|
|
|
|
auto it = metric_map.find(name);
|
|
|
|
if (it == metric_map.end())
|
|
|
|
return nullptr;
|
|
|
|
else
|
|
|
|
return it->second;
|
|
|
|
};
|
|
|
|
|
2020-12-21 21:47:10 +00:00
|
|
|
if (servers_to_start_before_tables)
|
2020-12-17 13:47:03 +00:00
|
|
|
{
|
2020-12-21 21:47:10 +00:00
|
|
|
for (const auto & server : *servers_to_start_before_tables)
|
|
|
|
{
|
|
|
|
if (const auto * name = get_metric_name(server.getPortName()))
|
|
|
|
new_values[name] = server.currentThreads();
|
|
|
|
}
|
2020-12-17 13:47:03 +00:00
|
|
|
}
|
|
|
|
|
2020-12-21 21:47:10 +00:00
|
|
|
if (servers)
|
2020-12-17 13:47:03 +00:00
|
|
|
{
|
2020-12-21 21:47:10 +00:00
|
|
|
for (const auto & server : *servers)
|
|
|
|
{
|
|
|
|
if (const auto * name = get_metric_name(server.getPortName()))
|
|
|
|
new_values[name] = server.currentThreads();
|
|
|
|
}
|
2020-12-17 13:47:03 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2016-10-23 06:12:50 +00:00
|
|
|
|
2020-04-16 12:31:57 +00:00
|
|
|
#if USE_JEMALLOC && JEMALLOC_VERSION_MAJOR >= 4
|
2020-06-18 01:54:10 +00:00
|
|
|
// 'epoch' is a special mallctl -- it updates the statistics. Without it, all
|
|
|
|
// the following calls will return stale values. It increments and returns
|
|
|
|
// the current epoch number, which might be useful to log as a sanity check.
|
|
|
|
auto epoch = updateJemallocEpoch();
|
|
|
|
new_values["jemalloc.epoch"] = epoch;
|
|
|
|
|
|
|
|
// Collect the statistics themselves.
|
|
|
|
saveJemallocMetric<size_t>(new_values, "allocated");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "active");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "metadata");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "metadata_thp");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "resident");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "mapped");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "retained");
|
|
|
|
saveJemallocMetric<size_t>(new_values, "background_thread.num_threads");
|
|
|
|
saveJemallocMetric<uint64_t>(new_values, "background_thread.num_runs");
|
|
|
|
saveJemallocMetric<uint64_t>(new_values, "background_thread.run_intervals");
|
|
|
|
saveAllArenasMetric<size_t>(new_values, "pactive");
|
|
|
|
saveAllArenasMetric<size_t>(new_values, "pdirty");
|
|
|
|
saveAllArenasMetric<size_t>(new_values, "pmuzzy");
|
|
|
|
saveAllArenasMetric<size_t>(new_values, "dirty_purged");
|
|
|
|
saveAllArenasMetric<size_t>(new_values, "muzzy_purged");
|
2018-08-02 00:20:20 +00:00
|
|
|
#endif
|
|
|
|
|
2020-06-28 23:44:24 +00:00
|
|
|
#if defined(OS_LINUX)
|
2020-06-25 20:36:50 +00:00
|
|
|
// Try to add processor frequencies, ignoring errors.
|
|
|
|
try
|
|
|
|
{
|
2020-06-26 00:16:58 +00:00
|
|
|
ReadBufferFromFile buf("/proc/cpuinfo", 32768 /* buf_size */);
|
2020-06-25 20:36:50 +00:00
|
|
|
|
|
|
|
// We need the following lines:
|
2020-09-17 10:47:29 +00:00
|
|
|
// processor : 4
|
2020-06-25 20:46:18 +00:00
|
|
|
// cpu MHz : 4052.941
|
2020-06-25 20:36:50 +00:00
|
|
|
// They contain tabs and are interspersed with other info.
|
|
|
|
int core_id = 0;
|
|
|
|
while (!buf.eof())
|
|
|
|
{
|
|
|
|
std::string s;
|
2020-06-26 00:16:58 +00:00
|
|
|
// We don't have any backslash escape sequences in /proc/cpuinfo, so
|
|
|
|
// this function will read the line until EOL, which is exactly what
|
|
|
|
// we need.
|
2020-06-25 20:36:50 +00:00
|
|
|
readEscapedStringUntilEOL(s, buf);
|
|
|
|
// It doesn't read the EOL itself.
|
|
|
|
++buf.position();
|
|
|
|
|
2020-09-17 10:47:29 +00:00
|
|
|
if (s.rfind("processor", 0) == 0)
|
2020-06-25 20:36:50 +00:00
|
|
|
{
|
|
|
|
if (auto colon = s.find_first_of(':'))
|
|
|
|
{
|
|
|
|
core_id = std::stoi(s.substr(colon + 2));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else if (s.rfind("cpu MHz", 0) == 0)
|
|
|
|
{
|
|
|
|
if (auto colon = s.find_first_of(':'))
|
|
|
|
{
|
|
|
|
auto mhz = std::stod(s.substr(colon + 2));
|
|
|
|
new_values[fmt::format("CPUFrequencyMHz_{}", core_id)] = mhz;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
tryLogCurrentException(__PRETTY_FUNCTION__);
|
|
|
|
}
|
2020-06-28 23:44:24 +00:00
|
|
|
#endif
|
2020-06-25 20:36:50 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Add more metrics as you wish.
|
2020-06-10 19:17:30 +00:00
|
|
|
|
|
|
|
// Log the new metrics.
|
2020-12-17 13:47:03 +00:00
|
|
|
if (auto log = global_context.getAsynchronousMetricLog())
|
2020-06-10 19:17:30 +00:00
|
|
|
{
|
|
|
|
log->addValues(new_values);
|
|
|
|
}
|
|
|
|
|
|
|
|
// Finally, update the current metrics.
|
|
|
|
std::lock_guard lock(mutex);
|
|
|
|
values = new_values;
|
2016-10-23 06:12:50 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
}
|