ClickHouse/dbms/src/Server/PerformanceTest.cpp

1493 lines
52 KiB
C++
Raw Normal View History

2017-05-07 18:55:42 +00:00
#include <functional>
2017-01-13 18:26:51 +00:00
#include <iostream>
#include <limits>
2017-05-07 18:55:42 +00:00
#include <regex>
#include <thread>
2017-02-24 22:02:08 +00:00
#include <unistd.h>
2017-01-13 18:26:51 +00:00
2017-05-05 18:32:38 +00:00
#include <boost/filesystem.hpp>
#include <boost/program_options.hpp>
2017-05-05 11:36:55 +00:00
#include <sys/stat.h>
2017-01-20 12:36:16 +00:00
#include <common/DateLUT.h>
2017-05-05 11:36:55 +00:00
#include <AggregateFunctions/ReservoirSampler.h>
2017-05-05 12:41:18 +00:00
#include <Client/Connection.h>
2017-05-05 11:36:55 +00:00
#include <Common/ConcurrentBoundedQueue.h>
#include <Common/Stopwatch.h>
#include <Common/getFQDNOrHostName.h>
#include <Common/getMultipleKeysFromConfig.h>
#include <Common/getNumberOfPhysicalCPUCores.h>
2017-05-05 11:36:55 +00:00
#include <Core/Types.h>
#include <DataStreams/RemoteBlockInputStream.h>
2017-05-06 14:36:37 +00:00
#include <IO/ReadBufferFromFile.h>
2017-05-05 11:36:55 +00:00
#include <IO/ReadHelpers.h>
2017-05-08 19:25:38 +00:00
#include <IO/WriteBufferFromFile.h>
2017-05-05 11:36:55 +00:00
#include <Interpreters/Settings.h>
#include <common/ThreadPool.h>
#include <common/getMemoryAmount.h>
2017-01-20 12:36:16 +00:00
#include <Poco/AutoPtr.h>
2017-05-05 11:36:55 +00:00
#include <Poco/Exception.h>
2017-01-20 12:36:16 +00:00
#include <Poco/SAX/InputSource.h>
#include <Poco/Util/XMLConfiguration.h>
2017-05-05 11:36:55 +00:00
#include <Poco/XML/XMLStream.h>
2017-01-20 12:36:16 +00:00
2017-02-24 22:02:08 +00:00
#include "InterruptListener.h"
2017-01-20 12:36:16 +00:00
2017-01-13 18:26:51 +00:00
/** Tests launcher for ClickHouse.
* The tool walks through given or default folder in order to find files with
2017-04-07 19:01:41 +00:00
* tests' descriptions and launches it.
2017-01-13 18:26:51 +00:00
*/
2017-05-06 14:36:37 +00:00
namespace FS = boost::filesystem;
2017-05-30 16:26:37 +00:00
using String = std::string;
const String FOUR_SPACES = " ";
2017-05-06 14:36:37 +00:00
2017-01-13 18:26:51 +00:00
namespace DB
{
namespace ErrorCodes
{
2017-05-05 11:36:55 +00:00
extern const int POCO_EXCEPTION;
extern const int STD_EXCEPTION;
extern const int UNKNOWN_EXCEPTION;
2017-06-15 19:39:35 +00:00
extern const int NOT_IMPLEMENTED;
2017-01-13 18:26:51 +00:00
}
static String pad(size_t padding)
{
2017-05-30 16:26:37 +00:00
return String(padding * 4, ' ');
2017-03-13 16:24:50 +00:00
}
2017-05-05 11:36:55 +00:00
class JSONString
{
private:
2017-05-30 16:26:37 +00:00
std::map<String, String> content;
size_t padding;
2017-05-05 11:36:55 +00:00
public:
2017-05-30 16:26:37 +00:00
JSONString(size_t padding_ = 1) : padding(padding_){};
2017-03-13 16:24:50 +00:00
2017-05-30 16:26:37 +00:00
void set(const String key, String value, bool wrap = true)
2017-03-13 16:24:50 +00:00
{
2017-05-05 11:36:55 +00:00
if (value.empty())
2017-03-13 16:24:50 +00:00
value = "null";
bool reserved = (value[0] == '[' || value[0] == '{' || value == "null");
2017-05-30 16:26:37 +00:00
if (!reserved && wrap)
2017-03-13 16:24:50 +00:00
value = '\"' + value + '\"';
2017-05-30 16:26:37 +00:00
content[key] = value;
2017-03-13 16:24:50 +00:00
}
2017-05-30 16:26:37 +00:00
template <typename T>
typename std::enable_if<std::is_arithmetic<T>::value>::type set(const String key, T value)
2017-03-13 16:24:50 +00:00
{
set(key, std::to_string(value), /*wrap= */ false);
2017-03-13 16:24:50 +00:00
}
2017-05-30 16:26:37 +00:00
void set(const String key, const std::vector<JSONString> & run_infos)
2017-03-13 16:24:50 +00:00
{
2017-05-30 16:26:37 +00:00
String value = "[\n";
2017-03-13 16:24:50 +00:00
2017-05-05 17:48:11 +00:00
for (size_t i = 0; i < run_infos.size(); ++i)
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
value += pad(padding + 1) + run_infos[i].asString(padding + 2);
2017-05-05 17:48:11 +00:00
if (i != run_infos.size() - 1)
2017-05-30 16:26:37 +00:00
value += ',';
2017-03-13 16:24:50 +00:00
2017-05-30 16:26:37 +00:00
value += "\n";
2017-03-13 16:24:50 +00:00
}
2017-05-30 16:26:37 +00:00
value += pad(padding) + ']';
content[key] = value;
2017-03-13 16:24:50 +00:00
}
String asString() const
{
return asString(padding);
}
2017-05-30 16:26:37 +00:00
String asString(size_t padding) const
2017-03-13 16:24:50 +00:00
{
2017-05-30 16:26:37 +00:00
String repr = "{";
2017-03-13 16:24:50 +00:00
2017-05-05 11:36:55 +00:00
for (auto it = content.begin(); it != content.end(); ++it)
{
2017-05-30 16:26:37 +00:00
if (it != content.begin())
repr += ',';
/// construct "key": "value" string with padding
repr += "\n" + pad(padding) + '\"' + it->first + '\"' + ": " + it->second;
2017-03-13 16:24:50 +00:00
}
2017-05-30 16:26:37 +00:00
repr += "\n" + pad(padding - 1) + '}';
return repr;
2017-03-13 16:24:50 +00:00
}
};
2017-03-27 18:25:42 +00:00
using ConfigurationPtr = Poco::AutoPtr<Poco::Util::AbstractConfiguration>;
/// A set of supported stop conditions.
struct StopConditionsSet
2017-05-05 11:36:55 +00:00
{
void loadFromConfig(const ConfigurationPtr & stop_conditions_view)
2017-05-05 11:36:55 +00:00
{
using Keys = std::vector<String>;
2017-05-05 11:36:55 +00:00
Keys keys;
stop_conditions_view->keys(keys);
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
for (const String & key : keys)
2017-05-05 11:36:55 +00:00
{
if (key == "total_time_ms")
total_time_ms.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else if (key == "rows_read")
rows_read.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else if (key == "bytes_read_uncompressed")
bytes_read_uncompressed.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else if (key == "iterations")
iterations.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else if (key == "min_time_not_changing_for_ms")
min_time_not_changing_for_ms.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else if (key == "max_speed_not_changing_for_ms")
max_speed_not_changing_for_ms.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else if (key == "average_speed_not_changing_for_ms")
average_speed_not_changing_for_ms.value = stop_conditions_view->getUInt64(key);
2017-05-05 11:36:55 +00:00
else
throw DB::Exception("Met unkown stop condition: " + key, 1);
2017-05-05 11:36:55 +00:00
++initialized_count;
2017-05-05 11:36:55 +00:00
}
}
void reset()
2017-05-05 11:36:55 +00:00
{
total_time_ms.fulfilled = false;
rows_read.fulfilled = false;
bytes_read_uncompressed.fulfilled = false;
iterations.fulfilled = false;
min_time_not_changing_for_ms.fulfilled = false;
max_speed_not_changing_for_ms.fulfilled = false;
average_speed_not_changing_for_ms.fulfilled = false;
fulfilled_count = 0;
2017-05-05 11:36:55 +00:00
}
/// Note: only conditions with UInt64 minimal thresholds are supported.
/// I.e. condition is fulfilled when value is exceeded.
struct StopCondition
2017-05-05 11:36:55 +00:00
{
UInt64 value = 0;
bool fulfilled = false;
};
2017-05-05 11:36:55 +00:00
void report(UInt64 value, StopCondition & condition)
2017-05-05 11:36:55 +00:00
{
if (condition.value && !condition.fulfilled && value >= condition.value)
2017-05-05 11:36:55 +00:00
{
condition.fulfilled = true;
++fulfilled_count;
2017-05-05 11:36:55 +00:00
}
}
StopCondition total_time_ms;
StopCondition rows_read;
StopCondition bytes_read_uncompressed;
StopCondition iterations;
StopCondition min_time_not_changing_for_ms;
StopCondition max_speed_not_changing_for_ms;
StopCondition average_speed_not_changing_for_ms;
2017-05-05 11:36:55 +00:00
size_t initialized_count = 0;
size_t fulfilled_count = 0;
};
/// Stop conditions for a test run. The running test will be terminated in either of two conditions:
/// 1. All conditions marked 'all_of' are fulfilled
/// or
/// 2. Any condition marked 'any_of' is fulfilled
class TestStopConditions
{
public:
void loadFromConfig(ConfigurationPtr & stop_conditions_config)
{
if (stop_conditions_config->has("all_of"))
{
ConfigurationPtr config_all_of(stop_conditions_config->createView("all_of"));
conditions_all_of.loadFromConfig(config_all_of);
}
if (stop_conditions_config->has("any_of"))
2017-05-05 11:36:55 +00:00
{
ConfigurationPtr config_any_of(stop_conditions_config->createView("any_of"));
conditions_any_of.loadFromConfig(config_any_of);
2017-05-05 11:36:55 +00:00
}
}
bool empty() const
2017-05-05 11:36:55 +00:00
{
return !conditions_all_of.initialized_count && !conditions_any_of.initialized_count;
}
#define DEFINE_REPORT_FUNC(FUNC_NAME, CONDITION) \
void FUNC_NAME(UInt64 value) \
{ \
conditions_all_of.report(value, conditions_all_of.CONDITION); \
conditions_any_of.report(value, conditions_any_of.CONDITION); \
}
DEFINE_REPORT_FUNC(reportTotalTime, total_time_ms);
DEFINE_REPORT_FUNC(reportRowsRead, rows_read);
DEFINE_REPORT_FUNC(reportBytesReadUncompressed, bytes_read_uncompressed);
DEFINE_REPORT_FUNC(reportIterations, iterations);
DEFINE_REPORT_FUNC(reportMinTimeNotChangingFor, min_time_not_changing_for_ms);
DEFINE_REPORT_FUNC(reportMaxSpeedNotChangingFor, max_speed_not_changing_for_ms);
DEFINE_REPORT_FUNC(reportAverageSpeedNotChangingFor, average_speed_not_changing_for_ms);
2017-05-05 11:36:55 +00:00
#undef REPORT
bool areFulfilled() const
{
return (conditions_all_of.initialized_count && conditions_all_of.fulfilled_count >= conditions_all_of.initialized_count)
|| (conditions_any_of.initialized_count && conditions_any_of.fulfilled_count);
}
void reset()
{
conditions_all_of.reset();
conditions_any_of.reset();
2017-05-05 11:36:55 +00:00
}
private:
StopConditionsSet conditions_all_of;
StopConditionsSet conditions_any_of;
};
struct Stats
{
2017-05-05 11:36:55 +00:00
Stopwatch watch;
Stopwatch watch_per_query;
Stopwatch min_time_watch;
Stopwatch max_rows_speed_watch;
Stopwatch max_bytes_speed_watch;
Stopwatch avg_rows_speed_watch;
Stopwatch avg_bytes_speed_watch;
bool last_query_was_cancelled = false;
size_t queries = 0;
size_t total_rows_read = 0;
size_t total_bytes_read = 0;
size_t last_query_rows_read = 0;
size_t last_query_bytes_read = 0;
2017-05-05 11:36:55 +00:00
using Sampler = ReservoirSampler<double>;
Sampler sampler{1 << 16};
/// min_time in ms
UInt64 min_time = std::numeric_limits<UInt64>::max();
double total_time = 0;
double max_rows_speed = 0;
double max_bytes_speed = 0;
double avg_rows_speed_value = 0;
double avg_rows_speed_first = 0;
static double avg_rows_speed_precision;
double avg_bytes_speed_value = 0;
double avg_bytes_speed_first = 0;
static double avg_bytes_speed_precision;
size_t number_of_rows_speed_info_batches = 0;
size_t number_of_bytes_speed_info_batches = 0;
2017-05-05 12:41:18 +00:00
bool ready = false; // check if a query wasn't interrupted by SIGINT
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
String getStatisticByName(const String & statistic_name)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
if (statistic_name == "min_time")
2017-05-05 11:36:55 +00:00
{
return std::to_string(min_time) + "ms";
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "quantiles")
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
String result = "\n";
2017-05-05 11:36:55 +00:00
for (double percent = 10; percent <= 90; percent += 10)
{
2017-05-05 17:48:11 +00:00
result += FOUR_SPACES + std::to_string((percent / 100));
2017-05-05 11:36:55 +00:00
result += ": " + std::to_string(sampler.quantileInterpolated(percent / 100.0));
result += "\n";
}
2017-05-05 17:48:11 +00:00
result += FOUR_SPACES + "0.95: " + std::to_string(sampler.quantileInterpolated(95 / 100.0)) + "\n";
result += FOUR_SPACES + "0.99: " + std::to_string(sampler.quantileInterpolated(99 / 100.0)) + "\n";
result += FOUR_SPACES + "0.999: " + std::to_string(sampler.quantileInterpolated(99.9 / 100.)) + "\n";
2017-05-05 17:48:11 +00:00
result += FOUR_SPACES + "0.9999: " + std::to_string(sampler.quantileInterpolated(99.99 / 100.));
2017-05-05 11:36:55 +00:00
return result;
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "total_time")
2017-05-05 11:36:55 +00:00
{
return std::to_string(total_time) + "s";
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "queries_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(queries / total_time);
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "rows_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(total_rows_read / total_time);
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "bytes_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(total_bytes_read / total_time);
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "max_rows_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(max_rows_speed);
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "max_bytes_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(max_bytes_speed);
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "avg_rows_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(avg_rows_speed_value);
}
2017-05-05 17:48:11 +00:00
if (statistic_name == "avg_bytes_per_second")
2017-05-05 11:36:55 +00:00
{
return std::to_string(avg_bytes_speed_value);
}
return "";
}
void update_min_time(const UInt64 min_time_candidate)
{
if (min_time_candidate < min_time)
{
min_time = min_time_candidate;
min_time_watch.restart();
}
}
void update_average_speed(const double new_speed_info,
Stopwatch & avg_speed_watch,
size_t & number_of_info_batches,
double precision,
double & avg_speed_first,
double & avg_speed_value)
{
avg_speed_value = ((avg_speed_value * number_of_info_batches) + new_speed_info);
avg_speed_value /= (++number_of_info_batches);
if (avg_speed_first == 0)
{
avg_speed_first = avg_speed_value;
}
if (abs(avg_speed_value - avg_speed_first) >= precision)
{
avg_speed_first = avg_speed_value;
avg_speed_watch.restart();
}
}
void update_max_speed(const size_t max_speed_candidate, Stopwatch & max_speed_watch, double & max_speed)
{
if (max_speed_candidate > max_speed)
{
max_speed = max_speed_candidate;
max_speed_watch.restart();
}
}
void add(size_t rows_read_inc, size_t bytes_read_inc)
{
total_rows_read += rows_read_inc;
total_bytes_read += bytes_read_inc;
last_query_rows_read += rows_read_inc;
last_query_bytes_read += bytes_read_inc;
2017-05-05 11:36:55 +00:00
double new_rows_speed = last_query_rows_read / watch_per_query.elapsedSeconds();
double new_bytes_speed = last_query_bytes_read / watch_per_query.elapsedSeconds();
2017-05-05 11:36:55 +00:00
/// Update rows speed
update_max_speed(new_rows_speed, max_rows_speed_watch, max_rows_speed);
update_average_speed(new_rows_speed,
avg_rows_speed_watch,
number_of_rows_speed_info_batches,
avg_rows_speed_precision,
avg_rows_speed_first,
avg_rows_speed_value);
/// Update bytes speed
update_max_speed(new_bytes_speed, max_bytes_speed_watch, max_bytes_speed);
update_average_speed(new_bytes_speed,
avg_bytes_speed_watch,
number_of_bytes_speed_info_batches,
avg_bytes_speed_precision,
avg_bytes_speed_first,
avg_bytes_speed_value);
}
void updateQueryInfo()
{
++queries;
sampler.insert(watch_per_query.elapsedSeconds());
update_min_time(watch_per_query.elapsed() / (1000 * 1000)); /// ns to ms
}
void setTotalTime()
{
total_time = watch.elapsedSeconds();
}
void clear()
{
watch.restart();
watch_per_query.restart();
min_time_watch.restart();
max_rows_speed_watch.restart();
max_bytes_speed_watch.restart();
avg_rows_speed_watch.restart();
avg_bytes_speed_watch.restart();
last_query_was_cancelled = false;
2017-05-05 11:36:55 +00:00
sampler.clear();
queries = 0;
total_rows_read = 0;
total_bytes_read = 0;
last_query_rows_read = 0;
last_query_bytes_read = 0;
2017-05-05 11:36:55 +00:00
min_time = std::numeric_limits<UInt64>::max();
total_time = 0;
max_rows_speed = 0;
max_bytes_speed = 0;
avg_rows_speed_value = 0;
avg_bytes_speed_value = 0;
avg_rows_speed_first = 0;
avg_bytes_speed_first = 0;
avg_rows_speed_precision = 0.001;
avg_bytes_speed_precision = 0.001;
number_of_rows_speed_info_batches = 0;
number_of_bytes_speed_info_batches = 0;
}
};
double Stats::avg_rows_speed_precision = 0.001;
2017-03-27 18:25:42 +00:00
double Stats::avg_bytes_speed_precision = 0.001;
2017-01-13 18:26:51 +00:00
class PerformanceTest
{
public:
2017-05-30 16:26:37 +00:00
using Strings = std::vector<String>;
2017-05-07 18:55:42 +00:00
PerformanceTest(const String & host_,
2017-05-05 11:36:55 +00:00
const UInt16 port_,
const String & default_database_,
const String & user_,
const String & password_,
2017-05-05 15:58:25 +00:00
const bool & lite_output_,
2017-05-30 16:26:37 +00:00
const String & profiles_file_,
2017-05-07 18:55:42 +00:00
Strings && input_files_,
Strings && tests_tags_,
Strings && skip_tags_,
Strings && tests_names_,
Strings && skip_names_,
Strings && tests_names_regexp_,
Strings && skip_names_regexp_)
2017-05-05 12:41:18 +00:00
: connection(host_, port_, default_database_, user_, password_),
2017-05-05 15:58:25 +00:00
gotSIGINT(false),
2017-05-07 00:29:30 +00:00
lite_output(lite_output_),
2017-05-07 18:55:42 +00:00
profiles_file(profiles_file_),
input_files(input_files_),
tests_tags(std::move(tests_tags_)),
skip_tags(std::move(skip_tags_)),
tests_names(std::move(tests_names_)),
skip_names(std::move(skip_names_)),
tests_names_regexp(std::move(tests_names_regexp_)),
skip_names_regexp(std::move(skip_names_regexp_))
2017-05-05 11:36:55 +00:00
{
if (input_files.size() < 1)
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("No tests were specified", 0);
2017-05-05 11:36:55 +00:00
}
std::string name;
UInt64 version_major;
UInt64 version_minor;
UInt64 version_revision;
connection.getServerVersion(name, version_major, version_minor, version_revision);
std::stringstream ss;
ss << version_major << "." << version_minor << "." << version_revision;
server_version = ss.str();
2017-05-07 18:55:42 +00:00
processTestsConfigurations(input_files);
2017-05-05 11:36:55 +00:00
}
2017-01-13 18:26:51 +00:00
private:
2017-05-30 16:26:37 +00:00
String test_name;
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
using Query = String;
2017-05-05 11:36:55 +00:00
using Queries = std::vector<Query>;
using QueriesWithIndexes = std::vector<std::pair<Query, size_t>>;
Queries queries;
2017-05-05 12:41:18 +00:00
Connection connection;
std::string server_version;
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
using Keys = std::vector<String>;
2017-05-05 11:36:55 +00:00
Settings settings;
Context global_context = Context::createGlobal();
2017-05-05 11:36:55 +00:00
InterruptListener interrupt_listener;
using XMLConfiguration = Poco::Util::XMLConfiguration;
using XMLConfigurationPtr = Poco::AutoPtr<XMLConfiguration>;
2017-05-07 18:55:42 +00:00
2017-05-30 16:26:37 +00:00
using Paths = std::vector<String>;
using StringToVector = std::map<String, std::vector<String>>;
2017-05-05 11:36:55 +00:00
StringToVector substitutions;
2017-05-30 16:26:37 +00:00
using StringKeyValue = std::map<String, String>;
2017-05-05 17:48:11 +00:00
std::vector<StringKeyValue> substitutions_maps;
2017-05-05 11:36:55 +00:00
2017-05-05 12:41:18 +00:00
bool gotSIGINT;
std::vector<TestStopConditions> stop_conditions_by_run;
2017-05-30 16:26:37 +00:00
String main_metric;
2017-05-05 15:58:25 +00:00
bool lite_output;
2017-05-30 16:26:37 +00:00
String profiles_file;
2017-05-05 11:36:55 +00:00
2017-05-07 18:55:42 +00:00
Strings input_files;
std::vector<XMLConfigurationPtr> tests_configurations;
2017-05-07 18:55:42 +00:00
Strings tests_tags;
Strings skip_tags;
Strings tests_names;
Strings skip_names;
Strings tests_names_regexp;
Strings skip_names_regexp;
2017-05-30 16:26:37 +00:00
enum class ExecutionType
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
Loop,
Once
2017-05-05 11:36:55 +00:00
};
2017-05-05 17:48:11 +00:00
ExecutionType exec_type;
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
enum class FilterType
2017-05-07 18:55:42 +00:00
{
2017-05-30 16:26:37 +00:00
Tag,
Name,
Name_regexp
2017-05-07 18:55:42 +00:00
};
2017-05-05 16:49:14 +00:00
size_t times_to_run = 1;
std::vector<Stats> statistics_by_run;
2017-05-05 11:36:55 +00:00
2017-05-07 18:55:42 +00:00
/// Removes configurations that has a given value. If leave is true, the logic is reversed.
void removeConfigurationsIf(
std::vector<XMLConfigurationPtr> & configs, FilterType filter_type, const Strings & values, bool leave = false)
2017-05-07 18:55:42 +00:00
{
auto checker = [&filter_type, &values, &leave](XMLConfigurationPtr & config) {
2017-05-07 18:55:42 +00:00
if (values.size() == 0)
return false;
bool remove_or_not = false;
2017-05-30 16:26:37 +00:00
if (filter_type == FilterType::Tag)
2017-05-07 18:55:42 +00:00
{
Keys tags_keys;
config->keys("tags", tags_keys);
Strings tags(tags_keys.size());
for (size_t i = 0; i != tags_keys.size(); ++i)
tags[i] = config->getString("tags.tag[" + std::to_string(i) + "]");
for (const String & config_tag : tags)
{
2017-05-07 18:55:42 +00:00
if (std::find(values.begin(), values.end(), config_tag) != values.end())
remove_or_not = true;
}
}
2017-05-30 16:26:37 +00:00
if (filter_type == FilterType::Name)
2017-05-07 18:55:42 +00:00
{
remove_or_not = (std::find(values.begin(), values.end(), config->getString("name", "")) != values.end());
}
2017-05-30 16:26:37 +00:00
if (filter_type == FilterType::Name_regexp)
2017-05-07 18:55:42 +00:00
{
2017-05-30 16:26:37 +00:00
String config_name = config->getString("name", "");
auto regex_checker = [&config_name](const String & name_regexp) {
2017-05-07 18:55:42 +00:00
std::regex pattern(name_regexp);
return std::regex_search(config_name, pattern);
};
remove_or_not = config->has("name") ? (std::find_if(values.begin(), values.end(), regex_checker) != values.end()) : false;
2017-05-07 18:55:42 +00:00
}
if (leave)
remove_or_not = !remove_or_not;
return remove_or_not;
};
auto new_end = std::remove_if(configs.begin(), configs.end(), checker);
2017-05-07 18:55:42 +00:00
configs.erase(new_end, configs.end());
}
/// Filter tests by tags, names, regexp matching, etc.
void filterConfigurations()
{
/// Leave tests:
2017-05-30 16:26:37 +00:00
removeConfigurationsIf(tests_configurations, FilterType::Tag, tests_tags, true);
removeConfigurationsIf(tests_configurations, FilterType::Name, tests_names, true);
removeConfigurationsIf(tests_configurations, FilterType::Name_regexp, tests_names_regexp, true);
2017-05-07 18:55:42 +00:00
/// Skip tests
2017-05-30 16:26:37 +00:00
removeConfigurationsIf(tests_configurations, FilterType::Tag, skip_tags, false);
removeConfigurationsIf(tests_configurations, FilterType::Name, skip_names, false);
removeConfigurationsIf(tests_configurations, FilterType::Name_regexp, skip_names_regexp, false);
2017-05-07 18:55:42 +00:00
}
2017-05-08 19:25:38 +00:00
/// Checks specified preconditions per test (process cache, table existence, etc.)
bool checkPreconditions(const XMLConfigurationPtr & config)
2017-05-08 19:25:38 +00:00
{
if (!config->has("preconditions"))
return true;
Keys preconditions;
config->keys("preconditions", preconditions);
size_t table_precondition_index = 0;
2017-05-30 16:26:37 +00:00
for (const String & precondition : preconditions)
2017-05-08 19:25:38 +00:00
{
if (precondition == "flush_disk_cache")
if (system(
"(>&2 echo 'Flushing disk cache...') && (sudo sh -c 'echo 3 > /proc/sys/vm/drop_caches') && (>&2 echo 'Flushed.')"))
{
std::cerr << "Failed to flush disk cache" << std::endl;
2017-05-30 16:26:37 +00:00
return false;
}
2017-05-08 19:25:38 +00:00
if (precondition == "ram_size")
{
size_t ram_size_needed = config->getUInt64("preconditions.ram_size");
size_t actual_ram = getMemoryAmount();
if (!actual_ram)
throw DB::Exception("ram_size precondition not available on this platform", ErrorCodes::NOT_IMPLEMENTED);
if (ram_size_needed > actual_ram)
2017-05-08 19:25:38 +00:00
{
std::cerr << "Not enough RAM: need = " << ram_size_needed << ", present = " << actual_ram << std::endl;
return false;
2017-05-08 19:25:38 +00:00
}
}
if (precondition == "table_exists")
{
2017-05-30 16:26:37 +00:00
String precondition_key = "preconditions.table_exists[" + std::to_string(table_precondition_index++) + "]";
String table_to_check = config->getString(precondition_key);
String query = "EXISTS TABLE " + table_to_check + ";";
2017-05-08 19:25:38 +00:00
size_t exist = 0;
connection.sendQuery(query, "", QueryProcessingStage::Complete, &settings, nullptr, false);
while (true)
{
Connection::Packet packet = connection.receivePacket();
if (packet.type == Protocol::Server::Data)
{
2017-05-08 19:25:38 +00:00
for (const ColumnWithTypeAndName & column : packet.block.getColumns())
{
if (column.name == "result" && column.column->size() > 0)
{
2017-05-08 19:25:38 +00:00
exist = column.column->get64(0);
if (exist)
break;
2017-05-08 19:25:38 +00:00
}
}
}
if (packet.type == Protocol::Server::Exception || packet.type == Protocol::Server::EndOfStream)
break;
}
if (!exist)
{
std::cerr << "Table " << table_to_check << " doesn't exist" << std::endl;
2017-05-08 19:25:38 +00:00
return false;
}
}
}
return true;
}
2017-05-07 18:55:42 +00:00
void processTestsConfigurations(const Paths & input_files)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
tests_configurations.resize(input_files.size());
2017-05-05 11:36:55 +00:00
for (size_t i = 0; i != input_files.size(); ++i)
{
2017-05-30 16:26:37 +00:00
const String path = input_files[i];
tests_configurations[i] = XMLConfigurationPtr(new XMLConfiguration(path));
2017-05-05 11:36:55 +00:00
}
2017-05-07 18:55:42 +00:00
filterConfigurations();
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
if (tests_configurations.size())
2017-05-05 11:36:55 +00:00
{
Strings outputs;
2017-05-05 17:48:11 +00:00
for (auto & test_config : tests_configurations)
2017-05-05 11:36:55 +00:00
{
2017-05-08 19:25:38 +00:00
if (!checkPreconditions(test_config))
{
std::cerr << "Preconditions are not fulfilled for test \"" + test_config->getString("name", "") + "\" ";
2017-05-08 19:25:38 +00:00
continue;
}
2017-05-30 16:26:37 +00:00
String output = runTest(test_config);
if (lite_output)
std::cout << output;
else
outputs.push_back(output);
}
if (!lite_output && outputs.size())
{
std::cout << "[" << std::endl;
for (size_t i = 0; i != outputs.size(); ++i)
{
std::cout << outputs[i];
if (i != outputs.size() - 1)
std::cout << ",";
std::cout << std::endl;
}
std::cout << "]" << std::endl;
2017-05-05 11:36:55 +00:00
}
}
}
void extractSettings(
const XMLConfigurationPtr & config, const String & key, const Strings & settings_list, std::map<String, String> & settings_to_apply)
2017-05-07 00:29:30 +00:00
{
2017-05-30 16:26:37 +00:00
for (const String & setup : settings_list)
2017-05-07 00:29:30 +00:00
{
if (setup == "profile")
continue;
2017-05-30 16:26:37 +00:00
String value = config->getString(key + "." + setup);
2017-05-07 00:29:30 +00:00
if (value.empty())
value = "true";
settings_to_apply[setup] = value;
}
}
String runTest(XMLConfigurationPtr & test_config)
2017-05-05 11:36:55 +00:00
{
2017-05-06 22:38:14 +00:00
queries.clear();
2017-05-05 17:48:11 +00:00
test_name = test_config->getString("name");
std::cerr << "Running: " << test_name << "\n";
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
if (test_config->has("settings"))
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
std::map<String, String> settings_to_apply;
2017-05-05 17:48:11 +00:00
Keys config_settings;
test_config->keys("settings", config_settings);
2017-05-05 11:36:55 +00:00
2017-05-07 00:29:30 +00:00
/// Preprocess configuration file
if (std::find(config_settings.begin(), config_settings.end(), "profile") != config_settings.end())
{
if (!profiles_file.empty())
{
2017-05-30 16:26:37 +00:00
String profile_name = test_config->getString("settings.profile");
XMLConfigurationPtr profiles_config(new XMLConfiguration(profiles_file));
2017-05-07 00:29:30 +00:00
Keys profile_settings;
profiles_config->keys("profiles." + profile_name, profile_settings);
extractSettings(profiles_config, "profiles." + profile_name, profile_settings, settings_to_apply);
}
}
extractSettings(test_config, "settings", config_settings, settings_to_apply);
2017-05-05 11:36:55 +00:00
/// This macro goes through all settings in the Settings.h
/// and, if found any settings in test's xml configuration
/// with the same name, sets its value to settings
2017-05-30 16:26:37 +00:00
std::map<String, String>::iterator it;
#define EXTRACT_SETTING(TYPE, NAME, DEFAULT) \
it = settings_to_apply.find(#NAME); \
if (it != settings_to_apply.end()) \
settings.set(#NAME, settings_to_apply[#NAME]);
2017-05-05 11:36:55 +00:00
2017-05-07 00:29:30 +00:00
APPLY_FOR_SETTINGS(EXTRACT_SETTING)
APPLY_FOR_LIMITS(EXTRACT_SETTING)
#undef EXTRACT_SETTING
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
if (std::find(config_settings.begin(), config_settings.end(), "average_rows_speed_precision") != config_settings.end())
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
Stats::avg_rows_speed_precision = test_config->getDouble("settings.average_rows_speed_precision");
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
if (std::find(config_settings.begin(), config_settings.end(), "average_bytes_speed_precision") != config_settings.end())
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
Stats::avg_bytes_speed_precision = test_config->getDouble("settings.average_bytes_speed_precision");
2017-05-05 11:36:55 +00:00
}
}
Query query;
2017-05-06 14:36:37 +00:00
if (!test_config->has("query") && !test_config->has("query_file"))
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("Missing query fields in test's config: " + test_name, 1);
2017-05-06 14:36:37 +00:00
}
if (test_config->has("query") && test_config->has("query_file"))
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("Found both query and query_file fields. Choose only one", 1);
2017-05-05 11:36:55 +00:00
}
2017-05-06 14:36:37 +00:00
if (test_config->has("query"))
{
queries = DB::getMultipleValuesFromConfig(*test_config, "", "query");
}
2017-05-06 14:36:37 +00:00
if (test_config->has("query_file"))
{
2017-05-30 16:26:37 +00:00
const String filename = test_config->getString("query_file");
2017-05-06 14:36:37 +00:00
if (filename.empty())
2017-05-30 16:26:37 +00:00
throw DB::Exception("Empty file name", 1);
2017-05-06 14:36:37 +00:00
bool tsv = FS::path(filename).extension().string() == ".tsv";
ReadBufferFromFile query_file(filename);
2017-05-06 22:25:18 +00:00
2017-06-13 06:21:52 +00:00
if (tsv)
{
while (!query_file.eof())
{
readEscapedString(query, query_file);
assertChar('\n', query_file);
2017-05-06 22:25:18 +00:00
queries.push_back(query);
2017-06-13 06:21:52 +00:00
}
}
else
{
readStringUntilEOF(query, query_file);
queries.push_back(query);
2017-05-06 14:36:37 +00:00
}
}
2017-05-05 11:36:55 +00:00
2017-05-06 14:36:37 +00:00
if (queries.empty())
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("Did not find any query to execute: " + test_name, 1);
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
if (test_config->has("substitutions"))
2017-05-05 11:36:55 +00:00
{
2017-05-06 22:38:14 +00:00
if (queries.size() > 1)
2017-05-30 16:26:37 +00:00
throw DB::Exception("Only one query is allowed when using substitutions", 1);
2017-05-06 22:38:14 +00:00
2017-05-05 11:36:55 +00:00
/// Make "subconfig" of inner xml block
ConfigurationPtr substitutions_view(test_config->createView("substitutions"));
2017-05-05 17:48:11 +00:00
constructSubstitutions(substitutions_view, substitutions);
2017-05-05 11:36:55 +00:00
2017-05-06 22:38:14 +00:00
queries = formatQueries(queries[0], substitutions);
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
if (!test_config->has("type"))
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("Missing type property in config: " + test_name, 1);
2017-05-05 11:36:55 +00:00
}
2017-05-30 16:26:37 +00:00
String config_exec_type = test_config->getString("type");
2017-05-05 17:48:11 +00:00
if (config_exec_type == "loop")
2017-05-30 16:26:37 +00:00
exec_type = ExecutionType::Loop;
2017-05-05 17:48:11 +00:00
else if (config_exec_type == "once")
2017-05-30 16:26:37 +00:00
exec_type = ExecutionType::Once;
2017-05-05 11:36:55 +00:00
else
2017-05-30 16:26:37 +00:00
throw DB::Exception("Unknown type " + config_exec_type + " in :" + test_name, 1);
2017-05-05 11:36:55 +00:00
2017-07-18 16:34:43 +00:00
times_to_run = test_config->getUInt("times_to_run", 1);
2017-05-05 11:36:55 +00:00
2017-07-17 18:01:47 +00:00
stop_conditions_by_run.clear();
TestStopConditions stop_conditions_template;
if (test_config->has("stop_conditions"))
2017-05-05 11:36:55 +00:00
{
ConfigurationPtr stop_conditions_config(test_config->createView("stop_conditions"));
stop_conditions_template.loadFromConfig(stop_conditions_config);
2017-05-05 11:36:55 +00:00
}
if (stop_conditions_template.empty())
2017-05-30 16:26:37 +00:00
throw DB::Exception("No termination conditions were found in config", 1);
2017-05-05 11:36:55 +00:00
for (size_t i = 0; i < times_to_run * queries.size(); ++i)
stop_conditions_by_run.push_back(stop_conditions_template);
ConfigurationPtr metrics_view(test_config->createView("metrics"));
2017-05-05 11:36:55 +00:00
Keys metrics;
2017-05-05 17:48:11 +00:00
metrics_view->keys(metrics);
2017-07-17 18:01:47 +00:00
main_metric.clear();
if (test_config->has("main_metric"))
{
Keys main_metrics;
test_config->keys("main_metric", main_metrics);
if (main_metrics.size())
main_metric = main_metrics[0];
}
2017-05-05 11:36:55 +00:00
2017-05-08 19:25:38 +00:00
if (!main_metric.empty())
{
2017-05-05 15:58:25 +00:00
if (std::find(metrics.begin(), metrics.end(), main_metric) == metrics.end())
metrics.push_back(main_metric);
2017-05-08 19:25:38 +00:00
}
else
{
2017-05-05 15:58:25 +00:00
if (lite_output)
2017-05-30 16:26:37 +00:00
throw DB::Exception("Specify main_metric for lite output", 1);
2017-05-05 11:36:55 +00:00
}
2017-05-05 15:58:25 +00:00
if (metrics.size() > 0)
checkMetricsInput(metrics);
2017-05-05 11:36:55 +00:00
statistics_by_run.resize(times_to_run * queries.size());
2017-05-05 17:48:11 +00:00
for (size_t number_of_launch = 0; number_of_launch < times_to_run; ++number_of_launch)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
QueriesWithIndexes queries_with_indexes;
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
for (size_t query_index = 0; query_index < queries.size(); ++query_index)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
size_t statistic_index = number_of_launch * queries.size() + query_index;
stop_conditions_by_run[statistic_index].reset();
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
queries_with_indexes.push_back({queries[query_index], statistic_index});
2017-05-05 11:36:55 +00:00
}
if (interrupt_listener.check())
gotSIGINT = true;
if (gotSIGINT)
break;
2017-05-05 17:48:11 +00:00
runQueries(queries_with_indexes);
2017-05-05 11:36:55 +00:00
}
2017-05-05 15:58:25 +00:00
if (lite_output)
return minOutput(main_metric);
2017-05-05 11:36:55 +00:00
else
2017-05-09 17:17:24 +00:00
return constructTotalInfo(metrics);
2017-05-05 11:36:55 +00:00
}
2017-05-05 15:58:25 +00:00
void checkMetricsInput(const Strings & metrics) const
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
std::vector<String> loop_metrics
2017-05-05 11:36:55 +00:00
= {"min_time", "quantiles", "total_time", "queries_per_second", "rows_per_second", "bytes_per_second"};
2017-05-30 16:26:37 +00:00
std::vector<String> non_loop_metrics
2017-05-05 11:36:55 +00:00
= {"max_rows_per_second", "max_bytes_per_second", "avg_rows_per_second", "avg_bytes_per_second"};
2017-05-30 16:26:37 +00:00
if (exec_type == ExecutionType::Loop)
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
for (const String & metric : metrics)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
if (std::find(non_loop_metrics.begin(), non_loop_metrics.end(), metric) != non_loop_metrics.end())
2017-05-05 12:41:18 +00:00
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("Wrong type of metric for loop execution type (" + metric + ")", 1);
2017-05-05 12:41:18 +00:00
}
2017-05-05 11:36:55 +00:00
}
}
else
{
2017-05-30 16:26:37 +00:00
for (const String & metric : metrics)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
if (std::find(loop_metrics.begin(), loop_metrics.end(), metric) != loop_metrics.end())
2017-05-05 12:41:18 +00:00
{
2017-05-30 16:26:37 +00:00
throw DB::Exception("Wrong type of metric for non-loop execution type (" + metric + ")", 1);
2017-05-05 12:41:18 +00:00
}
2017-05-05 11:36:55 +00:00
}
}
}
2017-05-05 17:48:11 +00:00
void runQueries(const QueriesWithIndexes & queries_with_indexes)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
for (const std::pair<Query, const size_t> & query_and_index : queries_with_indexes)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
Query query = query_and_index.first;
const size_t run_index = query_and_index.second;
TestStopConditions & stop_conditions = stop_conditions_by_run[run_index];
Stats & statistics = statistics_by_run[run_index];
2017-05-05 11:36:55 +00:00
size_t iteration = 0;
statistics.clear();
execute(query, statistics, stop_conditions);
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
if (exec_type == ExecutionType::Loop)
2017-05-05 11:36:55 +00:00
{
while (!gotSIGINT)
{
++iteration;
stop_conditions.reportIterations(iteration);
if (stop_conditions.areFulfilled())
2017-05-05 11:36:55 +00:00
break;
execute(query, statistics, stop_conditions);
2017-05-05 11:36:55 +00:00
}
}
if (!gotSIGINT)
{
statistics.ready = true;
2017-05-05 11:36:55 +00:00
}
}
}
void execute(const Query & query, Stats & statistics, TestStopConditions & stop_conditions)
2017-05-05 11:36:55 +00:00
{
statistics.watch_per_query.restart();
statistics.last_query_was_cancelled = false;
statistics.last_query_rows_read = 0;
statistics.last_query_bytes_read = 0;
2017-05-05 11:36:55 +00:00
RemoteBlockInputStream stream(connection, query, global_context, &settings);
2017-05-05 11:36:55 +00:00
stream.setProgressCallback(
[&](const Progress & value) { this->checkFulfilledConditionsAndUpdate(value, stream, statistics, stop_conditions); });
2017-05-05 11:36:55 +00:00
2017-05-05 12:41:18 +00:00
stream.readPrefix();
while (Block block = stream.read())
2017-05-05 11:36:55 +00:00
;
2017-05-05 12:41:18 +00:00
stream.readSuffix();
2017-05-05 11:36:55 +00:00
if (!statistics.last_query_was_cancelled)
statistics.updateQueryInfo();
statistics.setTotalTime();
2017-05-05 11:36:55 +00:00
}
void checkFulfilledConditionsAndUpdate(
const Progress & progress, RemoteBlockInputStream & stream, Stats & statistics, TestStopConditions & stop_conditions)
2017-05-05 11:36:55 +00:00
{
statistics.add(progress.rows, progress.bytes);
2017-05-05 11:36:55 +00:00
stop_conditions.reportRowsRead(statistics.total_rows_read);
stop_conditions.reportBytesReadUncompressed(statistics.total_bytes_read);
stop_conditions.reportTotalTime(statistics.watch.elapsed() / (1000 * 1000));
stop_conditions.reportMinTimeNotChangingFor(statistics.min_time_watch.elapsed() / (1000 * 1000));
stop_conditions.reportMaxSpeedNotChangingFor(statistics.max_rows_speed_watch.elapsed() / (1000 * 1000));
stop_conditions.reportAverageSpeedNotChangingFor(statistics.avg_rows_speed_watch.elapsed() / (1000 * 1000));
2017-05-05 11:36:55 +00:00
if (stop_conditions.areFulfilled())
2017-05-05 11:36:55 +00:00
{
statistics.last_query_was_cancelled = true;
2017-05-05 12:41:18 +00:00
stream.cancel();
2017-05-05 11:36:55 +00:00
}
2017-05-05 12:41:18 +00:00
if (interrupt_listener.check())
{
2017-05-05 11:36:55 +00:00
gotSIGINT = true;
statistics.last_query_was_cancelled = true;
2017-05-05 12:41:18 +00:00
stream.cancel();
2017-05-05 11:36:55 +00:00
}
}
void constructSubstitutions(ConfigurationPtr & substitutions_view, StringToVector & substitutions)
2017-05-05 11:36:55 +00:00
{
Keys xml_substitutions;
2017-05-05 17:48:11 +00:00
substitutions_view->keys(xml_substitutions);
2017-05-05 11:36:55 +00:00
for (size_t i = 0; i != xml_substitutions.size(); ++i)
{
const ConfigurationPtr xml_substitution(substitutions_view->createView("substitution[" + std::to_string(i) + "]"));
2017-05-05 11:36:55 +00:00
/// Property values for substitution will be stored in a vector
/// accessible by property name
2017-05-30 16:26:37 +00:00
std::vector<String> xml_values;
2017-05-05 11:36:55 +00:00
xml_substitution->keys("values", xml_values);
2017-05-30 16:26:37 +00:00
String name = xml_substitution->getString("name");
2017-05-05 11:36:55 +00:00
for (size_t j = 0; j != xml_values.size(); ++j)
{
substitutions[name].push_back(xml_substitution->getString("values.value[" + std::to_string(j) + "]"));
}
}
}
2017-05-30 16:26:37 +00:00
std::vector<String> formatQueries(const String & query, StringToVector substitutions)
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
std::vector<String> queries;
2017-05-05 11:36:55 +00:00
StringToVector::iterator substitutions_first = substitutions.begin();
StringToVector::iterator substitutions_last = substitutions.end();
--substitutions_last;
2017-05-30 16:26:37 +00:00
std::map<String, String> substitutions_map;
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
runThroughAllOptionsAndPush(substitutions_first, substitutions_last, query, queries, substitutions_map);
2017-05-05 11:36:55 +00:00
return queries;
}
/// Recursive method which goes through all substitution blocks in xml
/// and replaces property {names} by their values
void runThroughAllOptionsAndPush(StringToVector::iterator substitutions_left,
StringToVector::iterator substitutions_right,
2017-05-30 16:26:37 +00:00
const String & template_query,
std::vector<String> & queries,
2017-05-05 17:48:11 +00:00
const StringKeyValue & template_substitutions_map = StringKeyValue())
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
String name = substitutions_left->first;
std::vector<String> values = substitutions_left->second;
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
for (const String & value : values)
2017-05-05 11:36:55 +00:00
{
/// Copy query string for each unique permutation
Query query = template_query;
2017-05-05 17:48:11 +00:00
StringKeyValue substitutions_map = template_substitutions_map;
size_t substr_pos = 0;
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
while (substr_pos != String::npos)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
substr_pos = query.find("{" + name + "}");
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
if (substr_pos != String::npos)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
query.replace(substr_pos, 1 + name.length() + 1, value);
2017-05-05 11:36:55 +00:00
}
}
2017-05-05 17:48:11 +00:00
substitutions_map[name] = value;
2017-05-05 11:36:55 +00:00
/// If we've reached the end of substitution chain
if (substitutions_left == substitutions_right)
{
queries.push_back(query);
2017-05-05 17:48:11 +00:00
substitutions_maps.push_back(substitutions_map);
2017-05-05 11:36:55 +00:00
}
else
{
StringToVector::iterator next_it = substitutions_left;
++next_it;
2017-05-05 17:48:11 +00:00
runThroughAllOptionsAndPush(next_it, substitutions_right, query, queries, substitutions_map);
2017-05-05 11:36:55 +00:00
}
}
}
2017-02-24 22:02:08 +00:00
public:
2017-05-30 16:26:37 +00:00
String constructTotalInfo(Strings metrics)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
JSONString json_output;
2017-05-05 11:36:55 +00:00
json_output.set("hostname", getFQDNOrHostName());
json_output.set("num_cores", getNumberOfPhysicalCPUCores());
json_output.set("num_threads", std::thread::hardware_concurrency());
json_output.set("ram", getMemoryAmount());
json_output.set("server_version", server_version);
json_output.set("time", DateLUT::instance().timeToString(time(0)));
2017-05-30 16:26:37 +00:00
json_output.set("test_name", test_name);
json_output.set("main_metric", main_metric);
2017-05-05 11:36:55 +00:00
if (substitutions.size())
{
2017-05-05 17:48:11 +00:00
JSONString json_parameters(2); /// here, 2 is the size of \t padding
2017-05-05 11:36:55 +00:00
for (auto it = substitutions.begin(); it != substitutions.end(); ++it)
{
2017-05-30 16:26:37 +00:00
String parameter = it->first;
std::vector<String> values = it->second;
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
String array_string = "[";
2017-05-05 11:36:55 +00:00
for (size_t i = 0; i != values.size(); ++i)
{
2017-05-05 17:48:11 +00:00
array_string += '\"' + values[i] + '\"';
2017-05-05 11:36:55 +00:00
if (i != values.size() - 1)
{
2017-05-05 17:48:11 +00:00
array_string += ", ";
2017-05-05 11:36:55 +00:00
}
}
2017-05-05 17:48:11 +00:00
array_string += ']';
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
json_parameters.set(parameter, array_string);
2017-05-05 11:36:55 +00:00
}
2017-05-30 16:26:37 +00:00
json_output.set("parameters", json_parameters.asString());
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
std::vector<JSONString> run_infos;
for (size_t query_index = 0; query_index < queries.size(); ++query_index)
2017-05-05 11:36:55 +00:00
{
for (size_t number_of_launch = 0; number_of_launch < times_to_run; ++number_of_launch)
2017-05-05 11:36:55 +00:00
{
Stats & statistics = statistics_by_run[number_of_launch * queries.size() + query_index];
if (!statistics.ready)
2017-05-05 16:49:14 +00:00
continue;
JSONString runJSON;
2017-05-05 11:36:55 +00:00
runJSON.set("query", queries[query_index]);
2017-05-05 17:48:11 +00:00
if (substitutions_maps.size())
2017-05-05 11:36:55 +00:00
{
2017-05-05 16:49:14 +00:00
JSONString parameters(4);
2017-05-05 17:48:11 +00:00
for (auto it = substitutions_maps[query_index].begin(); it != substitutions_maps[query_index].end(); ++it)
2017-05-05 16:49:14 +00:00
{
2017-05-30 16:26:37 +00:00
parameters.set(it->first, it->second);
2017-05-05 16:49:14 +00:00
}
2017-05-30 16:26:37 +00:00
runJSON.set("parameters", parameters.asString());
2017-05-05 11:36:55 +00:00
}
2017-05-30 16:26:37 +00:00
if (exec_type == ExecutionType::Loop)
2017-05-05 16:49:14 +00:00
{
/// in seconds
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "min_time") != metrics.end())
runJSON.set("min_time", statistics.min_time / double(1000));
2017-05-05 11:36:55 +00:00
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "quantiles") != metrics.end())
2017-05-05 16:49:14 +00:00
{
2017-05-09 17:17:24 +00:00
JSONString quantiles(4); /// here, 4 is the size of \t padding
for (double percent = 10; percent <= 90; percent += 10)
{
2017-05-30 16:26:37 +00:00
String quantile_key = std::to_string(percent / 100.0);
while (quantile_key.back() == '0')
quantile_key.pop_back();
quantiles.set(quantile_key, statistics.sampler.quantileInterpolated(percent / 100.0));
2017-05-09 17:17:24 +00:00
}
quantiles.set("0.95", statistics.sampler.quantileInterpolated(95 / 100.0));
quantiles.set("0.99", statistics.sampler.quantileInterpolated(99 / 100.0));
quantiles.set("0.999", statistics.sampler.quantileInterpolated(99.9 / 100.0));
quantiles.set("0.9999", statistics.sampler.quantileInterpolated(99.99 / 100.0));
2017-05-09 17:17:24 +00:00
2017-05-30 16:26:37 +00:00
runJSON.set("quantiles", quantiles.asString());
2017-05-05 16:49:14 +00:00
}
2017-05-05 11:36:55 +00:00
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "total_time") != metrics.end())
runJSON.set("total_time", statistics.total_time);
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "queries_per_second") != metrics.end())
runJSON.set("queries_per_second", double(statistics.queries) / statistics.total_time);
2017-05-05 16:49:14 +00:00
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "rows_per_second") != metrics.end())
runJSON.set("rows_per_second", double(statistics.total_rows_read) / statistics.total_time);
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "bytes_per_second") != metrics.end())
runJSON.set("bytes_per_second", double(statistics.total_bytes_read) / statistics.total_time);
2017-05-05 16:49:14 +00:00
}
else
2017-05-05 11:36:55 +00:00
{
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "max_rows_per_second") != metrics.end())
runJSON.set("max_rows_per_second", statistics.max_rows_speed);
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "max_bytes_per_second") != metrics.end())
runJSON.set("max_bytes_per_second", statistics.max_bytes_speed);
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "avg_rows_per_second") != metrics.end())
runJSON.set("avg_rows_per_second", statistics.avg_rows_speed_value);
2017-05-09 17:17:24 +00:00
if (std::find(metrics.begin(), metrics.end(), "avg_bytes_per_second") != metrics.end())
runJSON.set("avg_bytes_per_second", statistics.avg_bytes_speed_value);
2017-05-05 11:36:55 +00:00
}
2017-05-05 17:48:11 +00:00
run_infos.push_back(runJSON);
2017-05-05 11:36:55 +00:00
}
}
2017-01-13 18:26:51 +00:00
2017-05-30 16:26:37 +00:00
json_output.set("runs", run_infos);
2017-05-05 11:36:55 +00:00
2017-05-30 16:26:37 +00:00
return json_output.asString();
2017-05-05 11:36:55 +00:00
}
2017-05-30 16:26:37 +00:00
String minOutput(const String & main_metric)
2017-05-05 11:36:55 +00:00
{
2017-05-30 16:26:37 +00:00
String output;
2017-05-05 17:48:11 +00:00
for (size_t query_index = 0; query_index < queries.size(); ++query_index)
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
for (size_t number_of_launch = 0; number_of_launch < times_to_run; ++number_of_launch)
2017-05-05 11:36:55 +00:00
{
if (queries.size() > 1)
{
output += "query \"" + queries[query_index] + "\", ";
}
2017-05-05 11:36:55 +00:00
2017-05-05 17:48:11 +00:00
if (substitutions_maps.size())
2017-05-05 11:36:55 +00:00
{
2017-05-05 17:48:11 +00:00
for (auto it = substitutions_maps[query_index].begin(); it != substitutions_maps[query_index].end(); ++it)
2017-05-05 11:36:55 +00:00
{
output += it->first + " = " + it->second + ", ";
2017-05-05 11:36:55 +00:00
}
}
output += "run " + std::to_string(number_of_launch + 1) + ": ";
output += main_metric + " = ";
output += statistics_by_run[number_of_launch * queries.size() + query_index].getStatisticByName(main_metric);
output += "\n";
2017-05-05 11:36:55 +00:00
}
}
return output;
2017-05-05 11:36:55 +00:00
}
};
2017-01-13 18:26:51 +00:00
}
static void getFilesFromDir(const FS::path & dir, std::vector<String> & input_files, const bool recursive = false)
2017-05-05 18:32:38 +00:00
{
if (dir.extension().string() == ".xml")
std::cerr << "Warning: \"" + dir.string() + "\" is a directory, but has .xml extension" << std::endl;
2017-05-06 14:36:37 +00:00
FS::directory_iterator end;
for (FS::directory_iterator it(dir); it != end; ++it)
2017-05-05 18:32:38 +00:00
{
2017-05-06 14:36:37 +00:00
const FS::path file = (*it);
if (recursive && FS::is_directory(file))
getFilesFromDir(file, input_files, recursive);
else if (!FS::is_directory(file) && file.extension().string() == ".xml")
2017-05-05 18:32:38 +00:00
input_files.push_back(file.string());
}
}
int mainEntryClickHousePerformanceTest(int argc, char ** argv)
2017-05-05 11:36:55 +00:00
{
using namespace DB;
try
{
using boost::program_options::value;
2017-05-30 16:26:37 +00:00
using Strings = std::vector<String>;
2017-05-05 11:36:55 +00:00
boost::program_options::options_description desc("Allowed options");
desc.add_options()("help", "produce help message")("lite", "use lite version of output")(
"profiles-file", value<String>()->default_value(""), "Specify a file with global profiles")(
"host,h", value<String>()->default_value("localhost"), "")("port", value<UInt16>()->default_value(9000), "")(
"database", value<String>()->default_value("default"), "")("user", value<String>()->default_value("default"), "")(
"password", value<String>()->default_value(""), "")("tags", value<Strings>()->multitoken(), "Run only tests with tag")(
"skip-tags", value<Strings>()->multitoken(), "Do not run tests with tag")("names",
value<Strings>()->multitoken(),
"Run tests with specific name")("skip-names", value<Strings>()->multitoken(), "Do not run tests with name")(
"names-regexp", value<Strings>()->multitoken(), "Run tests with names matching regexp")("skip-names-regexp",
value<Strings>()->multitoken(),
"Do not run tests with names matching regexp")("recursive,r", "Recurse in directories to find all xml's");
2017-05-05 11:36:55 +00:00
/// These options will not be displayed in --help
boost::program_options::options_description hidden("Hidden options");
2017-05-30 16:26:37 +00:00
hidden.add_options()("input-files", value<std::vector<String>>(), "");
2017-05-05 11:36:55 +00:00
/// But they will be legit, though. And they must be given without name
boost::program_options::positional_options_description positional;
positional.add("input-files", -1);
boost::program_options::options_description cmdline_options;
cmdline_options.add(desc).add(hidden);
boost::program_options::variables_map options;
boost::program_options::store(
boost::program_options::command_line_parser(argc, argv).options(cmdline_options).positional(positional).run(), options);
boost::program_options::notify(options);
if (options.count("help"))
{
std::cout << "Usage: " << argv[0] << " [options] [test_file ...] [tests_folder]\n";
std::cout << desc << "\n";
2017-05-05 15:58:25 +00:00
return 0;
2017-05-05 11:36:55 +00:00
}
2017-05-05 18:32:38 +00:00
Strings input_files;
bool recursive = options.count("recursive");
2017-05-05 18:32:38 +00:00
2017-05-05 11:36:55 +00:00
if (!options.count("input-files"))
{
std::cerr << "Trying to find test scenario files in the current folder...";
2017-05-30 16:26:37 +00:00
FS::path curr_dir(".");
2017-05-05 18:32:38 +00:00
getFilesFromDir(curr_dir, input_files, recursive);
2017-05-05 18:32:38 +00:00
if (input_files.empty())
{
std::cerr << std::endl;
2017-05-30 16:26:37 +00:00
throw DB::Exception("Did not find any xml files", 1);
}
else
std::cerr << " found " << input_files.size() << " files." << std::endl;
2017-05-08 19:25:38 +00:00
}
else
{
2017-05-05 18:32:38 +00:00
input_files = options["input-files"].as<Strings>();
2017-05-30 16:26:37 +00:00
for (const String filename : input_files)
2017-05-05 18:32:38 +00:00
{
2017-05-06 14:36:37 +00:00
FS::path file(filename);
2017-05-05 18:32:38 +00:00
2017-05-06 14:36:37 +00:00
if (!FS::exists(file))
2017-05-30 16:26:37 +00:00
throw DB::Exception("File \"" + filename + "\" does not exist", 1);
2017-05-05 18:32:38 +00:00
2017-05-06 14:36:37 +00:00
if (FS::is_directory(file))
2017-05-05 18:32:38 +00:00
{
input_files.erase(std::remove(input_files.begin(), input_files.end(), filename), input_files.end());
getFilesFromDir(file, input_files, recursive);
2017-05-08 19:25:38 +00:00
}
else
{
2017-05-05 18:32:38 +00:00
if (file.extension().string() != ".xml")
2017-05-30 16:26:37 +00:00
throw DB::Exception("File \"" + filename + "\" does not have .xml extension", 1);
2017-05-05 18:32:38 +00:00
}
}
2017-05-05 11:36:55 +00:00
}
Strings tests_tags = options.count("tags") ? options["tags"].as<Strings>() : Strings({});
Strings skip_tags = options.count("skip-tags") ? options["skip-tags"].as<Strings>() : Strings({});
Strings tests_names = options.count("names") ? options["names"].as<Strings>() : Strings({});
Strings skip_names = options.count("skip-names") ? options["skip-names"].as<Strings>() : Strings({});
Strings tests_names_regexp = options.count("names-regexp") ? options["names-regexp"].as<Strings>() : Strings({});
Strings skip_names_regexp = options.count("skip-names-regexp") ? options["skip-names-regexp"].as<Strings>() : Strings({});
PerformanceTest performanceTest(options["host"].as<String>(),
2017-05-05 11:36:55 +00:00
options["port"].as<UInt16>(),
2017-05-30 16:26:37 +00:00
options["database"].as<String>(),
options["user"].as<String>(),
options["password"].as<String>(),
2017-05-05 15:58:25 +00:00
options.count("lite") > 0,
2017-05-30 16:26:37 +00:00
options["profiles-file"].as<String>(),
2017-05-07 18:55:42 +00:00
std::move(input_files),
std::move(tests_tags),
std::move(skip_tags),
std::move(tests_names),
std::move(skip_names),
std::move(tests_names_regexp),
std::move(skip_names_regexp));
2017-05-05 11:36:55 +00:00
}
catch (...)
{
std::cout << getCurrentExceptionMessage(/*with stacktrace = */ true) << std::endl;
2017-05-30 16:26:37 +00:00
return getCurrentExceptionCode();
2017-05-05 11:36:55 +00:00
}
return 0;
2017-01-13 18:26:51 +00:00
}