Miscellaneous [#CLICKHOUSE-2]

This commit is contained in:
Alexey Milovidov 2018-02-25 05:43:27 +03:00
parent c86fd31097
commit 722b6287ec
6 changed files with 103 additions and 37 deletions

View File

@ -8,6 +8,18 @@
#include <common/apple_rt.h>
#endif
namespace StopWatchDetail
{
inline UInt64 nanoseconds(clockid_t clock_type)
{
struct timespec ts;
clock_gettime(clock_type, &ts);
return ts.tv_sec * 1000000000ULL + ts.tv_nsec;
}
}
/** Differs from Poco::Stopwatch only by using 'clock_gettime' instead of 'gettimeofday',
* returns nanoseconds instead of microseconds, and also by other minor differencies.
*/
@ -32,50 +44,49 @@ private:
clockid_t clock_type;
bool is_running;
UInt64 nanoseconds() const
{
struct timespec ts;
clock_gettime(clock_type, &ts);
return ts.tv_sec * 1000000000ULL + ts.tv_nsec;
}
UInt64 nanoseconds() const { return StopWatchDetail::nanoseconds(clock_type); }
};
class StopwatchWithLock : public Stopwatch
class AtomicStopwatch
{
public:
AtomicStopwatch(clockid_t clock_type_ = CLOCK_MONOTONIC) : clock_type(clock_type_) { restart(); }
void restart() { start_ns = nanoseconds(); }
UInt64 elapsed() const { return nanoseconds() - start_ns; }
UInt64 elapsedMilliseconds() const { return elapsed() / 1000000UL; }
double elapsedSeconds() const { return static_cast<double>(elapsed()) / 1000000000ULL; }
/** If specified amount of time has passed and timer is not locked right now, then restarts timer and returns true.
* Otherwise returns false.
* This is done atomically.
*/
bool lockTestAndRestart(double seconds)
bool compareAndRestart(double seconds)
{
std::unique_lock<std::mutex> lock(mutex, std::defer_lock);
if (!lock.try_lock())
return false;
UInt64 threshold = seconds * 1000000000ULL;
UInt64 current_ns = nanoseconds();
UInt64 current_start_ns = start_ns;
if (elapsedSeconds() >= seconds)
while (true)
{
restart();
return true;
if (current_ns < current_start_ns + threshold)
return false;
if (start_ns.compare_exchange_weak(current_start_ns, current_ns))
return true;
}
else
return false;
}
struct Lock
{
StopwatchWithLock * parent = nullptr;
std::unique_lock<std::mutex> lock;
AtomicStopwatch * parent = nullptr;
Lock() {}
operator bool() const { return parent != nullptr; }
Lock(StopwatchWithLock * parent, std::unique_lock<std::mutex> && lock)
: parent(parent), lock(std::move(lock))
{
}
Lock(AtomicStopwatch * parent) : parent(parent) {}
Lock(Lock &&) = default;
@ -92,21 +103,33 @@ public:
* This is done atomically.
*
* Usage:
* if (auto lock = timer.lockTestAndRestartAfter(1))
* if (auto lock = timer.compareAndRestartDeferred(1))
* /// do some work, that must be done in one thread and not more frequently than each second.
*/
Lock lockTestAndRestartAfter(double seconds)
Lock compareAndRestartDeferred(double seconds)
{
std::unique_lock<std::mutex> lock(mutex, std::defer_lock);
if (!lock.try_lock())
return {};
UInt64 threshold = seconds * 1000000000ULL;
UInt64 current_ns = nanoseconds();
UInt64 current_start_ns = start_ns;
if (elapsedSeconds() >= seconds)
return Lock(this, std::move(lock));
while (true)
{
if ((current_start_ns & 0x8000000000000000ULL))
return {};
return {};
if (current_ns < current_start_ns + threshold)
return {};
if (start_ns.compare_exchange_weak(current_start_ns, current_ns | 0x8000000000000000ULL))
return Lock(this);
}
}
private:
std::mutex mutex;
std::atomic<UInt64> start_ns;
std::atomic<bool> lock {false};
clockid_t clock_type;
/// Most significant bit is a lock. When it is set, compareAndRestartDeferred method will return false.
UInt64 nanoseconds() const { return StopWatchDetail::nanoseconds(clock_type) & 0x7FFFFFFFFFFFFFFFULL; }
};

View File

@ -68,3 +68,6 @@ target_link_libraries (allocator clickhouse_common_io)
add_executable (cow_columns cow_columns.cpp)
target_link_libraries (cow_columns clickhouse_common_io)
add_executable (stopwatch stopwatch.cpp)
target_link_libraries (stopwatch clickhouse_common_io)

View File

@ -0,0 +1,40 @@
#include <vector>
#include <thread>
#include <iostream>
#include <Common/Stopwatch.h>
int main(int, char **)
{
static constexpr size_t num_threads = 10;
static constexpr size_t num_iterations = 3;
std::vector<std::thread> threads(num_threads);
AtomicStopwatch watch;
Stopwatch total_watch;
for (size_t i = 0; i < num_threads; ++i)
{
threads[i] = std::thread([i, &watch, &total_watch]
{
size_t iteration = 0;
while (iteration < num_iterations)
{
if (auto lock = watch.compareAndRestartDeferred(1))
{
std::cerr << "Thread " << i << ": begin iteration " << iteration << ", elapsed: " << total_watch.elapsedMilliseconds() << " ms.\n";
std::this_thread::sleep_for(std::chrono::milliseconds(500));
std::cerr << "Thread " << i << ": end iteration " << iteration << ", elapsed: " << total_watch.elapsedMilliseconds() << " ms.\n";
++iteration;
}
std::this_thread::sleep_for(std::chrono::milliseconds(500));
}
});
}
for (auto & thread : threads)
thread.join();
return 0;
}

View File

@ -149,7 +149,7 @@ void DatabaseOrdinary::loadTables(
String data_path = context.getPath() + "data/" + escapeForFileName(name) + "/";
StopwatchWithLock watch;
AtomicStopwatch watch;
std::atomic<size_t> tables_processed {0};
auto task_function = [&](FileNames::const_iterator begin, FileNames::const_iterator end)
@ -160,7 +160,7 @@ void DatabaseOrdinary::loadTables(
/// Messages, so that it's not boring to wait for the server to load for a long time.
if ((++tables_processed) % PRINT_MESSAGE_EACH_N_TABLES == 0
|| watch.lockTestAndRestart(PRINT_MESSAGE_EACH_N_SECONDS))
|| watch.compareAndRestart(PRINT_MESSAGE_EACH_N_SECONDS))
{
LOG_INFO(log, std::fixed << std::setprecision(2) << tables_processed * 100.0 / total_tables << "%");
watch.restart();
@ -200,7 +200,7 @@ void DatabaseOrdinary::startupTables(ThreadPool * thread_pool)
{
LOG_INFO(log, "Starting up tables.");
StopwatchWithLock watch;
AtomicStopwatch watch;
std::atomic<size_t> tables_processed {0};
size_t total_tables = tables.size();
@ -209,7 +209,7 @@ void DatabaseOrdinary::startupTables(ThreadPool * thread_pool)
for (auto it = begin; it != end; ++it)
{
if ((++tables_processed) % PRINT_MESSAGE_EACH_N_TABLES == 0
|| watch.lockTestAndRestart(PRINT_MESSAGE_EACH_N_SECONDS))
|| watch.compareAndRestart(PRINT_MESSAGE_EACH_N_SECONDS))
{
LOG_INFO(log, std::fixed << std::setprecision(2) << tables_processed * 100.0 / total_tables << "%");
watch.restart();

View File

@ -293,7 +293,7 @@ bool StorageMergeTree::merge(
String * out_disable_reason)
{
/// Clear old parts. It does not matter to do it more frequently than each second.
if (auto lock = time_after_previous_cleanup.lockTestAndRestartAfter(1))
if (auto lock = time_after_previous_cleanup.compareAndRestartDeferred(1))
{
data.clearOldPartsFromFilesystem();
data.clearOldTemporaryDirectories();

View File

@ -100,7 +100,7 @@ private:
SimpleIncrement increment{0};
/// For clearOldParts, clearOldTemporaryDirectories.
StopwatchWithLock time_after_previous_cleanup;
AtomicStopwatch time_after_previous_cleanup;
MergeTreeData::DataParts currently_merging;
std::mutex currently_merging_mutex;