2014-03-10 09:33:18 +00:00
|
|
|
|
#include <unistd.h>
|
|
|
|
|
#include <stdlib.h>
|
|
|
|
|
#include <fcntl.h>
|
|
|
|
|
#include <signal.h>
|
2016-01-12 02:55:39 +00:00
|
|
|
|
#include <time.h>
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
#include <iostream>
|
|
|
|
|
#include <fstream>
|
|
|
|
|
#include <iomanip>
|
2016-01-12 02:55:39 +00:00
|
|
|
|
#include <random>
|
2016-09-28 16:49:59 +00:00
|
|
|
|
#include <limits.h>
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
#include <Poco/File.h>
|
|
|
|
|
#include <Poco/Util/Application.h>
|
|
|
|
|
|
2015-10-05 00:44:40 +00:00
|
|
|
|
#include <DB/Common/Stopwatch.h>
|
2016-08-02 01:46:05 +00:00
|
|
|
|
#include <DB/Common/ThreadPool.h>
|
2015-11-15 06:11:58 +00:00
|
|
|
|
#include <DB/AggregateFunctions/ReservoirSampler.h>
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
#include <boost/program_options.hpp>
|
|
|
|
|
|
|
|
|
|
#include <DB/Common/ConcurrentBoundedQueue.h>
|
|
|
|
|
|
2015-10-05 01:35:28 +00:00
|
|
|
|
#include <DB/Common/Exception.h>
|
2016-07-31 03:53:16 +00:00
|
|
|
|
#include <DB/Common/randomSeed.h>
|
2014-03-10 09:33:18 +00:00
|
|
|
|
#include <DB/Core/Types.h>
|
|
|
|
|
|
|
|
|
|
#include <DB/IO/ReadBufferFromFileDescriptor.h>
|
|
|
|
|
#include <DB/IO/WriteBufferFromFileDescriptor.h>
|
|
|
|
|
#include <DB/IO/ReadHelpers.h>
|
|
|
|
|
#include <DB/IO/WriteHelpers.h>
|
|
|
|
|
|
|
|
|
|
#include <DB/DataStreams/RemoteBlockInputStream.h>
|
|
|
|
|
|
|
|
|
|
#include <DB/Interpreters/Context.h>
|
|
|
|
|
|
|
|
|
|
#include <DB/Client/Connection.h>
|
|
|
|
|
|
|
|
|
|
#include "InterruptListener.h"
|
|
|
|
|
|
|
|
|
|
|
2016-09-28 16:49:59 +00:00
|
|
|
|
/** A tool for evaluating ClickHouse performance.
|
|
|
|
|
* The tool emulates a case with fixed amount of simultaneously executing queries.
|
2014-03-10 09:33:18 +00:00
|
|
|
|
*/
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
|
{
|
|
|
|
|
|
2016-01-12 02:55:39 +00:00
|
|
|
|
namespace ErrorCodes
|
|
|
|
|
{
|
|
|
|
|
extern const int POCO_EXCEPTION;
|
|
|
|
|
extern const int STD_EXCEPTION;
|
|
|
|
|
extern const int UNKNOWN_EXCEPTION;
|
|
|
|
|
}
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
class Benchmark
|
|
|
|
|
{
|
|
|
|
|
public:
|
2014-04-06 01:19:43 +00:00
|
|
|
|
Benchmark(unsigned concurrency_, double delay_,
|
2014-03-10 09:33:18 +00:00
|
|
|
|
const String & host_, UInt16 port_, const String & default_database_,
|
2015-12-26 01:04:12 +00:00
|
|
|
|
const String & user_, const String & password_, const String & stage,
|
2016-09-28 16:49:59 +00:00
|
|
|
|
bool randomize_, size_t max_iterations_, double max_time_,
|
|
|
|
|
const String & json_path_, const Settings & settings_)
|
|
|
|
|
:
|
|
|
|
|
concurrency(concurrency_), delay(delay_), queue(concurrency),
|
2015-05-28 03:49:28 +00:00
|
|
|
|
connections(concurrency, host_, port_, default_database_, user_, password_),
|
2016-09-28 16:49:59 +00:00
|
|
|
|
randomize(randomize_), max_iterations(max_iterations_), max_time(max_time_),
|
|
|
|
|
json_path(json_path_), settings(settings_), pool(concurrency)
|
2014-03-10 09:33:18 +00:00
|
|
|
|
{
|
|
|
|
|
std::cerr << std::fixed << std::setprecision(3);
|
|
|
|
|
|
2015-12-26 01:04:12 +00:00
|
|
|
|
if (stage == "complete")
|
|
|
|
|
query_processing_stage = QueryProcessingStage::Complete;
|
|
|
|
|
else if (stage == "fetch_columns")
|
|
|
|
|
query_processing_stage = QueryProcessingStage::FetchColumns;
|
|
|
|
|
else if (stage == "with_mergeable_state")
|
|
|
|
|
query_processing_stage = QueryProcessingStage::WithMergeableState;
|
|
|
|
|
else
|
|
|
|
|
throw Exception("Unknown query processing stage: " + stage, ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
2016-09-28 16:49:59 +00:00
|
|
|
|
if (!json_path.empty() && Poco::File(json_path).exists()) /// Clear file with previous results
|
|
|
|
|
{
|
|
|
|
|
Poco::File(json_path).remove();
|
|
|
|
|
}
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
readQueries();
|
|
|
|
|
run();
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
private:
|
2016-05-28 10:35:44 +00:00
|
|
|
|
using Query = std::string;
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
unsigned concurrency;
|
2014-04-06 01:19:43 +00:00
|
|
|
|
double delay;
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2016-05-28 10:35:44 +00:00
|
|
|
|
using Queries = std::vector<Query>;
|
2014-03-10 09:33:18 +00:00
|
|
|
|
Queries queries;
|
|
|
|
|
|
2016-05-28 10:35:44 +00:00
|
|
|
|
using Queue = ConcurrentBoundedQueue<Query>;
|
2014-03-10 09:33:18 +00:00
|
|
|
|
Queue queue;
|
|
|
|
|
|
|
|
|
|
ConnectionPool connections;
|
2016-01-12 02:55:39 +00:00
|
|
|
|
bool randomize;
|
2016-09-28 16:49:59 +00:00
|
|
|
|
size_t max_iterations;
|
|
|
|
|
double max_time;
|
|
|
|
|
String json_path;
|
2015-02-16 20:12:59 +00:00
|
|
|
|
Settings settings;
|
2015-12-26 01:04:12 +00:00
|
|
|
|
QueryProcessingStage::Enum query_processing_stage;
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2014-05-06 18:02:57 +00:00
|
|
|
|
struct Stats
|
|
|
|
|
{
|
|
|
|
|
Stopwatch watch;
|
|
|
|
|
size_t queries = 0;
|
|
|
|
|
size_t read_rows = 0;
|
|
|
|
|
size_t read_bytes = 0;
|
|
|
|
|
size_t result_rows = 0;
|
|
|
|
|
size_t result_bytes = 0;
|
2014-05-06 17:08:51 +00:00
|
|
|
|
|
2016-05-28 10:35:44 +00:00
|
|
|
|
using Sampler = ReservoirSampler<double>;
|
2014-05-06 18:02:57 +00:00
|
|
|
|
Sampler sampler {1 << 16};
|
|
|
|
|
|
|
|
|
|
void add(double seconds, size_t read_rows_inc, size_t read_bytes_inc, size_t result_rows_inc, size_t result_bytes_inc)
|
|
|
|
|
{
|
|
|
|
|
++queries;
|
|
|
|
|
read_rows += read_rows_inc;
|
|
|
|
|
read_bytes += read_bytes_inc;
|
|
|
|
|
result_rows += result_rows_inc;
|
|
|
|
|
result_bytes += result_bytes_inc;
|
|
|
|
|
sampler.insert(seconds);
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
void clear()
|
|
|
|
|
{
|
|
|
|
|
watch.restart();
|
|
|
|
|
queries = 0;
|
|
|
|
|
read_rows = 0;
|
|
|
|
|
read_bytes = 0;
|
|
|
|
|
result_rows = 0;
|
|
|
|
|
result_bytes = 0;
|
|
|
|
|
sampler.clear();
|
|
|
|
|
}
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
Stats info_per_interval;
|
|
|
|
|
Stats info_total;
|
2014-05-06 17:08:51 +00:00
|
|
|
|
|
2016-05-28 10:15:36 +00:00
|
|
|
|
std::mutex mutex;
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2016-08-02 01:46:05 +00:00
|
|
|
|
ThreadPool pool;
|
2015-03-16 19:24:57 +00:00
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
void readQueries()
|
|
|
|
|
{
|
|
|
|
|
ReadBufferFromFileDescriptor in(STDIN_FILENO);
|
|
|
|
|
|
|
|
|
|
while (!in.eof())
|
|
|
|
|
{
|
|
|
|
|
std::string query;
|
|
|
|
|
readText(query, in);
|
2016-02-07 08:42:21 +00:00
|
|
|
|
assertChar('\n', in);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
if (!query.empty())
|
|
|
|
|
queries.emplace_back(query);
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
if (queries.empty())
|
|
|
|
|
throw Exception("Empty list of queries.");
|
|
|
|
|
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "Loaded " << queries.size() << " queries.\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
2015-12-26 03:42:16 +00:00
|
|
|
|
void printNumberOfQueriesExecuted(size_t num)
|
|
|
|
|
{
|
|
|
|
|
std::cerr << "\nQueries executed: " << num;
|
|
|
|
|
if (queries.size() > 1)
|
|
|
|
|
std::cerr << " (" << (num * 100.0 / queries.size()) << "%)";
|
|
|
|
|
std::cerr << ".\n";
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
void run()
|
|
|
|
|
{
|
2016-07-31 03:53:16 +00:00
|
|
|
|
std::mt19937 generator(randomSeed());
|
2016-01-12 02:55:39 +00:00
|
|
|
|
std::uniform_int_distribution<size_t> distribution(0, queries.size() - 1);
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
for (size_t i = 0; i < concurrency; ++i)
|
2015-10-14 12:19:55 +00:00
|
|
|
|
pool.schedule(std::bind(&Benchmark::thread, this, connections.IConnectionPool::get()));
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
InterruptListener interrupt_listener;
|
|
|
|
|
|
2014-05-06 18:02:57 +00:00
|
|
|
|
info_per_interval.watch.restart();
|
2014-03-10 09:33:18 +00:00
|
|
|
|
Stopwatch watch;
|
|
|
|
|
|
|
|
|
|
/// В цикле, кладём все запросы в очередь.
|
2016-09-28 16:49:59 +00:00
|
|
|
|
for (size_t i = 0; !(max_iterations > 0) || i < max_iterations; ++i)
|
2014-03-10 09:33:18 +00:00
|
|
|
|
{
|
2016-09-28 16:49:59 +00:00
|
|
|
|
size_t query_index = randomize ? distribution(generator) : i % queries.size();
|
2016-01-12 02:55:39 +00:00
|
|
|
|
|
|
|
|
|
queue.push(queries[query_index]);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2016-09-28 16:49:59 +00:00
|
|
|
|
if (delay > 0 && watch.elapsedSeconds() > delay)
|
2014-03-10 09:33:18 +00:00
|
|
|
|
{
|
2015-12-26 03:42:16 +00:00
|
|
|
|
auto total_queries = 0;
|
|
|
|
|
{
|
2016-05-28 10:15:36 +00:00
|
|
|
|
std::lock_guard<std::mutex> lock(mutex);
|
2015-12-26 03:42:16 +00:00
|
|
|
|
total_queries = info_total.queries;
|
|
|
|
|
}
|
|
|
|
|
printNumberOfQueriesExecuted(total_queries);
|
|
|
|
|
|
2014-05-06 18:02:57 +00:00
|
|
|
|
report(info_per_interval);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
watch.restart();
|
|
|
|
|
}
|
2016-09-28 16:49:59 +00:00
|
|
|
|
|
|
|
|
|
if (max_time > 0 && info_total.watch.elapsedSeconds() >= max_time)
|
|
|
|
|
{
|
|
|
|
|
std::cout << "Stopping launch of queries. Requested time limit is exhausted.\n";
|
|
|
|
|
break;
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
if (interrupt_listener.check())
|
|
|
|
|
{
|
|
|
|
|
std::cout << "Stopping launch of queries. SIGINT recieved.\n";
|
|
|
|
|
break;
|
|
|
|
|
}
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/// Попросим потоки завершиться.
|
|
|
|
|
for (size_t i = 0; i < concurrency; ++i)
|
|
|
|
|
queue.push("");
|
|
|
|
|
|
|
|
|
|
pool.wait();
|
2014-05-06 17:08:51 +00:00
|
|
|
|
|
2016-09-28 16:49:59 +00:00
|
|
|
|
info_total.watch.stop();
|
|
|
|
|
if (!json_path.empty())
|
|
|
|
|
reportJSON(info_total, json_path);
|
2015-12-26 03:42:16 +00:00
|
|
|
|
printNumberOfQueriesExecuted(info_total.queries);
|
2014-05-06 18:02:57 +00:00
|
|
|
|
report(info_total);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
void thread(ConnectionPool::Entry connection)
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
Query query;
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
try
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
try
|
2014-03-10 09:33:18 +00:00
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
/// В этих потоках не будем принимать сигнал INT.
|
|
|
|
|
sigset_t sig_set;
|
|
|
|
|
if (sigemptyset(&sig_set)
|
|
|
|
|
|| sigaddset(&sig_set, SIGINT)
|
|
|
|
|
|| pthread_sigmask(SIG_BLOCK, &sig_set, nullptr))
|
|
|
|
|
throwFromErrno("Cannot block signal.", ErrorCodes::CANNOT_BLOCK_SIGNAL);
|
|
|
|
|
|
|
|
|
|
while (true)
|
|
|
|
|
{
|
|
|
|
|
queue.pop(query);
|
|
|
|
|
|
|
|
|
|
/// Пустой запрос обозначает конец работы.
|
|
|
|
|
if (query.empty())
|
|
|
|
|
break;
|
|
|
|
|
|
|
|
|
|
execute(connection, query);
|
|
|
|
|
}
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
2015-12-26 03:28:03 +00:00
|
|
|
|
catch (const Exception & e)
|
|
|
|
|
{
|
|
|
|
|
std::string text = e.displayText();
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "Code: " << e.code() << ". " << text << "\n\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2015-12-26 03:28:03 +00:00
|
|
|
|
/// Если есть стек-трейс на сервере, то не будем писать стек-трейс на клиенте.
|
|
|
|
|
if (std::string::npos == text.find("Stack trace"))
|
|
|
|
|
std::cerr << "Stack trace:\n"
|
|
|
|
|
<< e.getStackTrace().toString();
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2015-12-26 03:28:03 +00:00
|
|
|
|
throw;
|
|
|
|
|
}
|
|
|
|
|
catch (const Poco::Exception & e)
|
|
|
|
|
{
|
|
|
|
|
std::cerr << "Poco::Exception: " << e.displayText() << "\n";
|
|
|
|
|
throw;
|
|
|
|
|
}
|
|
|
|
|
catch (const std::exception & e)
|
|
|
|
|
{
|
|
|
|
|
std::cerr << "std::exception: " << e.what() << "\n";
|
|
|
|
|
throw;
|
|
|
|
|
}
|
|
|
|
|
catch (...)
|
|
|
|
|
{
|
|
|
|
|
std::cerr << "Unknown exception\n";
|
|
|
|
|
throw;
|
|
|
|
|
}
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
|
|
|
|
catch (...)
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "On query:\n" << query << "\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
throw;
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
void execute(ConnectionPool::Entry & connection, Query & query)
|
|
|
|
|
{
|
|
|
|
|
Stopwatch watch;
|
2015-12-26 01:04:12 +00:00
|
|
|
|
RemoteBlockInputStream stream(connection, query, &settings, nullptr, Tables(), query_processing_stage);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2014-10-25 18:33:52 +00:00
|
|
|
|
Progress progress;
|
|
|
|
|
stream.setProgressCallback([&progress](const Progress & value) { progress.incrementPiecewiseAtomically(value); });
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
stream.readPrefix();
|
|
|
|
|
while (Block block = stream.read())
|
|
|
|
|
;
|
|
|
|
|
stream.readSuffix();
|
|
|
|
|
|
2016-08-13 02:21:17 +00:00
|
|
|
|
const BlockStreamProfileInfo & info = stream.getProfileInfo();
|
2014-05-06 17:34:22 +00:00
|
|
|
|
|
2014-05-06 18:02:57 +00:00
|
|
|
|
double seconds = watch.elapsedSeconds();
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2016-05-28 10:15:36 +00:00
|
|
|
|
std::lock_guard<std::mutex> lock(mutex);
|
2014-10-25 18:33:52 +00:00
|
|
|
|
info_per_interval.add(seconds, progress.rows, progress.bytes, info.rows, info.bytes);
|
|
|
|
|
info_total.add(seconds, progress.rows, progress.bytes, info.rows, info.bytes);
|
2014-05-06 17:08:51 +00:00
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
2014-05-06 18:02:57 +00:00
|
|
|
|
void report(Stats & info)
|
2014-03-10 09:33:18 +00:00
|
|
|
|
{
|
2016-05-28 10:15:36 +00:00
|
|
|
|
std::lock_guard<std::mutex> lock(mutex);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2014-05-06 18:02:57 +00:00
|
|
|
|
double seconds = info.watch.elapsedSeconds();
|
2014-05-06 17:34:22 +00:00
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
std::cerr
|
2015-12-26 03:28:03 +00:00
|
|
|
|
<< "\n"
|
2014-05-06 18:02:57 +00:00
|
|
|
|
<< "QPS: " << (info.queries / seconds) << ", "
|
|
|
|
|
<< "RPS: " << (info.read_rows / seconds) << ", "
|
|
|
|
|
<< "MiB/s: " << (info.read_bytes / seconds / 1048576) << ", "
|
|
|
|
|
<< "result RPS: " << (info.result_rows / seconds) << ", "
|
|
|
|
|
<< "result MiB/s: " << (info.result_bytes / seconds / 1048576) << "."
|
2015-12-26 03:28:03 +00:00
|
|
|
|
<< "\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
2016-09-28 16:49:59 +00:00
|
|
|
|
auto print_percentile = [&](double percent)
|
|
|
|
|
{
|
2014-05-06 18:02:57 +00:00
|
|
|
|
std::cerr << percent << "%\t" << info.sampler.quantileInterpolated(percent / 100.0) << " sec." << std::endl;
|
2016-09-28 16:49:59 +00:00
|
|
|
|
};
|
2014-04-06 23:18:07 +00:00
|
|
|
|
|
2016-09-28 16:49:59 +00:00
|
|
|
|
for (int percent = 0; percent <= 90; percent += 10)
|
|
|
|
|
print_percentile(percent);
|
|
|
|
|
print_percentile(95);
|
|
|
|
|
print_percentile(99);
|
|
|
|
|
print_percentile(99.9);
|
|
|
|
|
print_percentile(99.99);
|
2014-05-06 18:02:57 +00:00
|
|
|
|
|
|
|
|
|
info.clear();
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
2016-09-28 16:49:59 +00:00
|
|
|
|
|
|
|
|
|
void reportJSON(Stats & info, const std::string & filename)
|
|
|
|
|
{
|
|
|
|
|
std::ofstream jout(filename);
|
|
|
|
|
if (!jout.is_open())
|
|
|
|
|
throw Exception("Can't write JSON data");
|
|
|
|
|
|
|
|
|
|
std::lock_guard<std::mutex> lock(mutex);
|
|
|
|
|
|
|
|
|
|
double seconds = info.watch.elapsedSeconds();
|
|
|
|
|
|
|
|
|
|
jout << "{\n";
|
|
|
|
|
|
|
|
|
|
jout << "\"statistics\": {\n"
|
|
|
|
|
<< "\"QPS\": " << (info.queries / seconds) << ",\n"
|
|
|
|
|
<< "\"RPS\": " << (info.read_rows / seconds) << ",\n"
|
|
|
|
|
<< "\"MiBPS\": " << (info.read_bytes / seconds / 1048576) << ",\n"
|
|
|
|
|
<< "\"RPS_result\": " << (info.result_rows / seconds) << ",\n"
|
|
|
|
|
<< "\"MiBPS_result\": " << (info.result_bytes / seconds / 1048576) << ",\n"
|
|
|
|
|
<< "\"num_queries\": " << info.queries << "\n"
|
|
|
|
|
<< "},\n";
|
|
|
|
|
|
|
|
|
|
auto print_percentile = [&](auto percent, bool with_comma = true)
|
|
|
|
|
{
|
|
|
|
|
jout << "\"" << percent << "\":\t" << info.sampler.quantileInterpolated(percent / 100.0) << (with_comma ? ",\n" : "\n");
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
jout << "\"query_time_percentiles\": {\n";
|
|
|
|
|
for (int percent = 0; percent <= 90; percent += 10)
|
|
|
|
|
print_percentile(percent);
|
|
|
|
|
print_percentile(95);
|
|
|
|
|
print_percentile(99);
|
|
|
|
|
print_percentile(99.9);
|
|
|
|
|
print_percentile(99.99, false);
|
|
|
|
|
jout << "}\n";
|
|
|
|
|
|
|
|
|
|
jout << "}\n";
|
|
|
|
|
|
|
|
|
|
jout.close();
|
|
|
|
|
}
|
2014-03-10 09:33:18 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
int main(int argc, char ** argv)
|
|
|
|
|
{
|
|
|
|
|
using namespace DB;
|
|
|
|
|
|
|
|
|
|
try
|
|
|
|
|
{
|
2016-09-28 16:49:59 +00:00
|
|
|
|
using boost::program_options::value;
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
boost::program_options::options_description desc("Allowed options");
|
|
|
|
|
desc.add_options()
|
2016-09-28 16:49:59 +00:00
|
|
|
|
("help", "produce help message")
|
|
|
|
|
("concurrency,c", value<unsigned>()->default_value(1), "number of parallel queries")
|
|
|
|
|
("delay,d", value<double>()->default_value(1), "delay between intermediate reports in seconds (set 0 to disable reports)")
|
|
|
|
|
("stage", value<std::string>()->default_value("complete"), "request query processing up to specified stage")
|
|
|
|
|
("iterations,i", value<size_t>()->default_value(0), "amount of queries to be executed")
|
|
|
|
|
("timelimit,t", value<double>()->default_value(0.), "stop launch of queries after specified time limit")
|
|
|
|
|
("randomize,r", value<bool>()->default_value(false), "randomize order of execution")
|
|
|
|
|
("json", value<std::string>()->default_value(""), "write final report to specified file in JSON format")
|
|
|
|
|
("host,h", value<std::string>()->default_value("localhost"), "")
|
|
|
|
|
("port", value<UInt16>()->default_value(9000), "")
|
|
|
|
|
("user", value<std::string>()->default_value("default"), "")
|
|
|
|
|
("password", value<std::string>()->default_value(""), "")
|
|
|
|
|
("database", value<std::string>()->default_value("default"), "")
|
|
|
|
|
|
2015-02-16 20:12:59 +00:00
|
|
|
|
#define DECLARE_SETTING(TYPE, NAME, DEFAULT) (#NAME, boost::program_options::value<std::string> (), "Settings.h")
|
|
|
|
|
#define DECLARE_LIMIT(TYPE, NAME, DEFAULT) (#NAME, boost::program_options::value<std::string> (), "Limits.h")
|
|
|
|
|
APPLY_FOR_SETTINGS(DECLARE_SETTING)
|
|
|
|
|
APPLY_FOR_LIMITS(DECLARE_LIMIT)
|
|
|
|
|
#undef DECLARE_SETTING
|
|
|
|
|
#undef DECLARE_LIMIT
|
2014-03-10 09:33:18 +00:00
|
|
|
|
;
|
|
|
|
|
|
|
|
|
|
boost::program_options::variables_map options;
|
|
|
|
|
boost::program_options::store(boost::program_options::parse_command_line(argc, argv, desc), options);
|
|
|
|
|
|
|
|
|
|
if (options.count("help"))
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cout << "Usage: " << argv[0] << " [options] < queries.txt\n";
|
|
|
|
|
std::cout << desc << "\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
return 1;
|
|
|
|
|
}
|
|
|
|
|
|
2015-02-16 20:12:59 +00:00
|
|
|
|
/// Извлекаем settings and limits из полученных options
|
|
|
|
|
Settings settings;
|
|
|
|
|
|
|
|
|
|
#define EXTRACT_SETTING(TYPE, NAME, DEFAULT) \
|
|
|
|
|
if (options.count(#NAME)) \
|
|
|
|
|
settings.set(#NAME, options[#NAME].as<std::string>());
|
|
|
|
|
APPLY_FOR_SETTINGS(EXTRACT_SETTING)
|
|
|
|
|
APPLY_FOR_LIMITS(EXTRACT_SETTING)
|
|
|
|
|
#undef EXTRACT_SETTING
|
|
|
|
|
|
2014-03-10 09:33:18 +00:00
|
|
|
|
Benchmark benchmark(
|
|
|
|
|
options["concurrency"].as<unsigned>(),
|
2014-04-06 01:19:43 +00:00
|
|
|
|
options["delay"].as<double>(),
|
2014-03-10 09:33:18 +00:00
|
|
|
|
options["host"].as<std::string>(),
|
|
|
|
|
options["port"].as<UInt16>(),
|
|
|
|
|
options["database"].as<std::string>(),
|
|
|
|
|
options["user"].as<std::string>(),
|
2015-02-16 20:12:59 +00:00
|
|
|
|
options["password"].as<std::string>(),
|
2015-12-26 01:04:12 +00:00
|
|
|
|
options["stage"].as<std::string>(),
|
2016-01-12 02:55:39 +00:00
|
|
|
|
options["randomize"].as<bool>(),
|
2016-09-28 16:49:59 +00:00
|
|
|
|
options["iterations"].as<size_t>(),
|
|
|
|
|
options["timelimit"].as<double>(),
|
|
|
|
|
options["json"].as<std::string>(),
|
2015-02-16 20:12:59 +00:00
|
|
|
|
settings);
|
2014-03-10 09:33:18 +00:00
|
|
|
|
}
|
|
|
|
|
catch (const Exception & e)
|
|
|
|
|
{
|
|
|
|
|
std::string text = e.displayText();
|
|
|
|
|
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "Code: " << e.code() << ". " << text << "\n\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
|
|
|
|
|
/// Если есть стек-трейс на сервере, то не будем писать стек-трейс на клиенте.
|
|
|
|
|
if (std::string::npos == text.find("Stack trace"))
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "Stack trace:\n"
|
2014-03-10 09:33:18 +00:00
|
|
|
|
<< e.getStackTrace().toString();
|
|
|
|
|
|
|
|
|
|
return e.code();
|
|
|
|
|
}
|
|
|
|
|
catch (const Poco::Exception & e)
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "Poco::Exception: " << e.displayText() << "\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
return ErrorCodes::POCO_EXCEPTION;
|
|
|
|
|
}
|
|
|
|
|
catch (const std::exception & e)
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "std::exception: " << e.what() << "\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
return ErrorCodes::STD_EXCEPTION;
|
|
|
|
|
}
|
|
|
|
|
catch (...)
|
|
|
|
|
{
|
2015-12-26 03:28:03 +00:00
|
|
|
|
std::cerr << "Unknown exception\n";
|
2014-03-10 09:33:18 +00:00
|
|
|
|
return ErrorCodes::UNKNOWN_EXCEPTION;
|
|
|
|
|
}
|
|
|
|
|
}
|