mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-29 19:12:03 +00:00
add setting to enable planning
This commit is contained in:
parent
0619b0921f
commit
fed6c65858
@ -668,6 +668,7 @@ class IColumn;
|
||||
M(Bool, mutations_execute_nondeterministic_on_initiator, false, "If true nondeterministic function are executed on initiator and replaced to literals in UPDATE and DELETE queries", 0) \
|
||||
M(Bool, mutations_execute_subqueries_on_initiator, false, "If true scalar subqueries are executed on initiator and replaced to literals in UPDATE and DELETE queries", 0) \
|
||||
M(UInt64, mutations_max_literal_size_to_replace, 16384, "The maximum size of serialized literal in bytes to replace in UPDATE and DELETE queries", 0) \
|
||||
M(Bool, allow_insert_threads_reduction_optimizaion, false, "If true it allows to apply additional single-insert-transformer for insertion of data", 0) \
|
||||
\
|
||||
M(Float, create_replicated_merge_tree_fault_injection_probability, 0.0f, "The probability of a fault injection during table creation after creating metadata in ZooKeeper", 0) \
|
||||
\
|
||||
|
@ -620,19 +620,32 @@ BlockIO InterpreterInsertQuery::execute()
|
||||
{
|
||||
bool table_prefers_large_blocks = table->prefersLargeBlocks();
|
||||
|
||||
pipeline.addTransform(std::make_shared<PlanSquashingTransform>(
|
||||
header,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL,
|
||||
presink_chains.size()));
|
||||
|
||||
pipeline.addSimpleTransform([&](const Block & in_header) -> ProcessorPtr
|
||||
if (settings.allow_insert_threads_reduction_optimizaion)
|
||||
{
|
||||
return std::make_shared<ApplySquashingTransform>(
|
||||
in_header,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL);
|
||||
});
|
||||
pipeline.addTransform(std::make_shared<PlanSquashingTransform>(
|
||||
header,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL,
|
||||
presink_chains.size()));
|
||||
|
||||
pipeline.addSimpleTransform([&](const Block & in_header) -> ProcessorPtr
|
||||
{
|
||||
return std::make_shared<ApplySquashingTransform>(
|
||||
in_header,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL);
|
||||
});
|
||||
}
|
||||
else
|
||||
{
|
||||
pipeline.addSimpleTransform([&](const Block & in_header) -> ProcessorPtr
|
||||
{
|
||||
return std::make_shared<SimpleSquashingTransform>(
|
||||
in_header,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL);
|
||||
});
|
||||
}
|
||||
}
|
||||
|
||||
size_t num_select_threads = pipeline.getNumThreads();
|
||||
@ -684,20 +697,33 @@ BlockIO InterpreterInsertQuery::execute()
|
||||
{
|
||||
bool table_prefers_large_blocks = table->prefersLargeBlocks();
|
||||
|
||||
auto squashing = std::make_shared<ApplySquashingTransform>(
|
||||
chain.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL);
|
||||
|
||||
chain.addSource(std::move(squashing));
|
||||
|
||||
auto balancing = std::make_shared<PlanSquashingTransform>(
|
||||
if (settings.allow_insert_threads_reduction_optimizaion)
|
||||
{
|
||||
auto squashing = std::make_shared<ApplySquashingTransform>(
|
||||
chain.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL,
|
||||
presink_chains.size());
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL);
|
||||
|
||||
chain.addSource(std::move(balancing));
|
||||
chain.addSource(std::move(squashing));
|
||||
|
||||
auto balancing = std::make_shared<PlanSquashingTransform>(
|
||||
chain.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL,
|
||||
presink_chains.size());
|
||||
|
||||
chain.addSource(std::move(balancing));
|
||||
}
|
||||
else
|
||||
{
|
||||
auto squashing = std::make_shared<SimpleSquashingTransform>(
|
||||
chain.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL);
|
||||
|
||||
chain.addSource(std::move(squashing));
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
auto context_ptr = getContext();
|
||||
|
@ -372,16 +372,26 @@ std::optional<Chain> generateViewChain(
|
||||
bool table_prefers_large_blocks = inner_table->prefersLargeBlocks();
|
||||
const auto & settings = insert_context->getSettingsRef();
|
||||
|
||||
out.addSource(std::make_shared<ApplySquashingTransform>(
|
||||
out.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL));
|
||||
if (settings.allow_insert_threads_reduction_optimizaion)
|
||||
{
|
||||
out.addSource(std::make_shared<ApplySquashingTransform>(
|
||||
out.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL));
|
||||
|
||||
out.addSource(std::make_shared<PlanSquashingTransform>(
|
||||
out.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL,
|
||||
1)); // Chain requires a single input
|
||||
out.addSource(std::make_shared<PlanSquashingTransform>(
|
||||
out.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL,
|
||||
1)); // Chain requires a single input
|
||||
}
|
||||
else
|
||||
{
|
||||
out.addSource(std::make_shared<SimpleSquashingTransform>(
|
||||
out.getInputHeader(),
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_rows : settings.max_block_size,
|
||||
table_prefers_large_blocks ? settings.min_insert_block_size_bytes : 0ULL));
|
||||
}
|
||||
}
|
||||
|
||||
auto counting = std::make_shared<CountingTransform>(out.getInputHeader(), current_thread, insert_context->getQuota());
|
||||
|
@ -11,6 +11,7 @@
|
||||
#include <Poco/Net/NetException.h>
|
||||
#include <Poco/Net/SocketAddress.h>
|
||||
#include <Poco/Util/LayeredConfiguration.h>
|
||||
#include <Common/escapeString.h>
|
||||
#include <Common/CurrentThread.h>
|
||||
#include <Common/Stopwatch.h>
|
||||
#include <Common/NetException.h>
|
||||
@ -42,6 +43,7 @@
|
||||
#include <Common/logger_useful.h>
|
||||
#include <Common/CurrentMetrics.h>
|
||||
#include <Common/thread_local_rng.h>
|
||||
#include <Access/User.h>
|
||||
#include <fmt/format.h>
|
||||
|
||||
#include <Processors/Executors/PullingAsyncPipelineExecutor.h>
|
||||
@ -181,6 +183,7 @@ void validateClientInfo(const ClientInfo & session_client_info, const ClientInfo
|
||||
|
||||
namespace DB
|
||||
{
|
||||
using Which = Field::Types::Which;
|
||||
|
||||
TCPHandler::TCPHandler(
|
||||
IServer & server_,
|
||||
@ -1602,6 +1605,70 @@ void TCPHandler::sendHello()
|
||||
nonce.emplace(thread_local_rng());
|
||||
writeIntBinary(nonce.value(), *out);
|
||||
}
|
||||
|
||||
/// If client is Clickhouse-client we will send server profile settings of this user
|
||||
if (client_name == (std::string(VERSION_NAME) + " client"))
|
||||
{
|
||||
const auto & user = session->sessionContext()->getUser();
|
||||
String query = fmt::format(
|
||||
R"(SELECT setting_name, value FROM system.settings_profile_elements WHERE user_name = '{0}')",
|
||||
escapeString(user->getName()));
|
||||
const auto & res_const = executeQuery(query,server.context(), QueryFlags{ .internal = true }).second;
|
||||
auto & res = const_cast<BlockIO &>(res_const);
|
||||
PullingPipelineExecutor pulling_executor(res.pipeline);
|
||||
Block block;
|
||||
pulling_executor.pull(block);
|
||||
/// filter data
|
||||
std::map<String, Field> server_settings;
|
||||
for (size_t row = 0; row < block.rows(); ++row)
|
||||
{
|
||||
size_t col_index = 0;
|
||||
String name;
|
||||
Field value_field;
|
||||
for (const auto & name_value: block)
|
||||
{
|
||||
Field field;
|
||||
name_value.column->get(row, field);
|
||||
if (!field.isNull())
|
||||
{
|
||||
if (col_index == 0)
|
||||
name = field.safeGet<String>();
|
||||
else
|
||||
value_field = field;
|
||||
}
|
||||
else
|
||||
continue;
|
||||
|
||||
++col_index;
|
||||
}
|
||||
if (!name.empty())
|
||||
server_settings[name] = value_field;
|
||||
|
||||
}
|
||||
|
||||
writeVarUInt(server_settings.size(), *out);
|
||||
if (!server_settings.empty())
|
||||
{
|
||||
for (const auto & setting : server_settings)
|
||||
{
|
||||
writeStringBinary(setting.first, *out);
|
||||
writeVarUInt(setting.second.getType(), *out);
|
||||
switch (setting.second.getType())
|
||||
{
|
||||
case Which::UInt64:
|
||||
writeVarUInt(setting.second.safeGet<UInt64>(), *out);break;
|
||||
case Which::String:
|
||||
writeStringBinary(setting.second.safeGet<String>(), *out);break;
|
||||
case Which::Bool:
|
||||
writeVarUInt(setting.second.get<UInt64>(), *out);break;
|
||||
default:
|
||||
break;
|
||||
}
|
||||
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
out->next();
|
||||
}
|
||||
|
||||
|
@ -311,8 +311,15 @@ Block ProjectionDescription::calculate(const Block & block, ContextPtr context)
|
||||
builder.resize(1);
|
||||
// Generate aggregated blocks with rows less or equal than the original block.
|
||||
// There should be only one output block after this transformation.
|
||||
builder.addTransform(std::make_shared<PlanSquashingTransform>(builder.getHeader(), block.rows(), 0, 1));
|
||||
builder.addTransform(std::make_shared<ApplySquashingTransform>(builder.getHeader(), block.rows(), 0));
|
||||
if (mut_context->getSettings().allow_insert_threads_reduction_optimizaion)
|
||||
{
|
||||
builder.addTransform(std::make_shared<PlanSquashingTransform>(builder.getHeader(), block.rows(), 0, 1));
|
||||
builder.addTransform(std::make_shared<ApplySquashingTransform>(builder.getHeader(), block.rows(), 0));
|
||||
}
|
||||
else
|
||||
{
|
||||
builder.addTransform(std::make_shared<SimpleSquashingTransform>(builder.getHeader(), block.rows(), 0));
|
||||
}
|
||||
|
||||
auto pipeline = QueryPipelineBuilder::getPipeline(std::move(builder));
|
||||
PullingPipelineExecutor executor(pipeline);
|
||||
|
Loading…
Reference in New Issue
Block a user