mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-24 00:22:29 +00:00
table functions: implementation of function remote, updates in cluster and storageDistributed [METR-9750]
This commit is contained in:
parent
9a12e98414
commit
325cc47ca5
@ -18,6 +18,9 @@ class Cluster
|
||||
public:
|
||||
Cluster(const Settings & settings, const DataTypeFactory & data_type_factory, const std::string & cluster_name);
|
||||
|
||||
/// Построить кластер по именам шардов и реплик, локальные обрабатываются так же как удаленные
|
||||
Cluster(const Settings & settings, const DataTypeFactory & data_type_factory, std::vector< std::vector<String> > names);
|
||||
|
||||
/// количество узлов clickhouse сервера, расположенных локально
|
||||
/// к локальным узлам обращаемся напрямую
|
||||
size_t getLocalNodesNum() const { return local_nodes_num; }
|
||||
|
@ -30,6 +30,17 @@ public:
|
||||
Context & context_,
|
||||
const String & sign_column_name_ = "");
|
||||
|
||||
static StoragePtr create(
|
||||
const std::string & name_, /// Имя таблицы.
|
||||
NamesAndTypesListPtr columns_, /// Список столбцов.
|
||||
const String & remote_database_, /// БД на удалённых серверах.
|
||||
const String & remote_table_, /// Имя таблицы на удалённых серверах.
|
||||
Cluster & cluster_,
|
||||
const DataTypeFactory & data_type_factory_,
|
||||
const Settings & settings,
|
||||
Context & context_,
|
||||
const String & sign_column_name_ = "");
|
||||
|
||||
std::string getName() const { return "Distributed"; }
|
||||
std::string getTableName() const { return name; }
|
||||
std::string getSignColumnName() const { return sign_column_name; };
|
||||
|
@ -27,27 +27,7 @@ public:
|
||||
virtual std::string getName() const = 0;
|
||||
|
||||
/// Создать storage в соответствии с запросом
|
||||
virtual StoragePtr execute(ASTPtr ast_function, Context & context) const = 0;
|
||||
|
||||
protected:
|
||||
/// Сгенерировать уникальное имя для временной таблицы.
|
||||
std::string chooseName () const {
|
||||
String result = "TemproraryTable" + getName() + "Id";
|
||||
for (size_t i = 0; i < 10; ++i)
|
||||
{
|
||||
int x = rand() % 62;
|
||||
char now;
|
||||
if (x < 10)
|
||||
now = '0' + rand() % 10;
|
||||
else if (x < 36)
|
||||
now = 'a' + x - 10;
|
||||
else
|
||||
now = 'A' + x - 36;
|
||||
|
||||
result += now;
|
||||
}
|
||||
return result;
|
||||
}
|
||||
virtual StoragePtr execute(ASTPtr ast_function, Context & context) = 0;
|
||||
};
|
||||
|
||||
typedef SharedPtr<ITableFunction> TableFunctionPtr;
|
||||
|
@ -24,7 +24,7 @@ class TableFunctionMerge: public ITableFunction
|
||||
public:
|
||||
std::string getName() const { return "merge"; }
|
||||
|
||||
StoragePtr execute(ASTPtr ast_function, Context & context) const override
|
||||
StoragePtr execute(ASTPtr ast_function, Context & context)
|
||||
{
|
||||
ASTs & args_func = dynamic_cast<ASTFunction &>(*ast_function).children;
|
||||
|
||||
@ -47,7 +47,7 @@ public:
|
||||
/// Нам необходимо его пометить как имя базы данных, посколку по умолчанию стоит значение column
|
||||
dynamic_cast<ASTIdentifier &>(*args[0]).kind = ASTIdentifier::Database;
|
||||
|
||||
return StorageMerge::create(chooseName(), chooseColumns(source_database, table_name_regexp, context), source_database, table_name_regexp, context);
|
||||
return StorageMerge::create(getName(), chooseColumns(source_database, table_name_regexp, context), source_database, table_name_regexp, context);
|
||||
}
|
||||
|
||||
private:
|
||||
|
@ -1,7 +1,11 @@
|
||||
#pragma once
|
||||
|
||||
#include <DB/TableFunctions/ITableFunction.h>
|
||||
#include <DB/Storages/StorageDistributed.h>
|
||||
#include <DB/Parsers/ASTIdentifier.h>
|
||||
#include <DB/DataStreams/RemoteBlockInputStream.h>
|
||||
|
||||
struct data;
|
||||
namespace DB
|
||||
{
|
||||
|
||||
@ -16,13 +20,221 @@ namespace DB
|
||||
class TableFunctionRemote: public ITableFunction
|
||||
{
|
||||
public:
|
||||
/// Максимальное количество различных шардов и максимальное количество реплик одного шарда
|
||||
const size_t MAX_ADDRESSES = 200;
|
||||
|
||||
std::string getName() const { return "remote"; }
|
||||
|
||||
StoragePtr execute(ASTPtr ast_function, Context & context) const override
|
||||
StoragePtr execute(ASTPtr ast_function, Context & context)
|
||||
{
|
||||
return StoragePtr();
|
||||
|
||||
/** В запросе в качестве аргумента для движка указано имя конфигурационной секции,
|
||||
* в которой задан список удалённых серверов, а также имя удалённой БД и имя удалённой таблицы.
|
||||
*/
|
||||
ASTs & args_func = dynamic_cast<ASTFunction &>(*ast_function).children;
|
||||
|
||||
if (args_func.size() != 1)
|
||||
throw Exception("Storage Distributed requires 3 parameters"
|
||||
" - name of configuration section with list of remote servers, name of remote database, name of remote table.",
|
||||
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
|
||||
|
||||
ASTs & args = dynamic_cast<ASTExpressionList &>(*args_func.at(0)).children;
|
||||
|
||||
if (args.size() != 3)
|
||||
throw Exception("Storage Distributed requires 3 parameters"
|
||||
" - description of remote servers, name of remote database, name of remote table.",
|
||||
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
|
||||
|
||||
String descripton = safeGet<const String &>(dynamic_cast<ASTLiteral &>(*args[0]).value);
|
||||
String remote_database = dynamic_cast<ASTIdentifier &>(*args[1]).name;
|
||||
String remote_table = dynamic_cast<ASTIdentifier &>(*args[2]).name;
|
||||
|
||||
/// В InterpreterSelectQuery будет создан ExpressionAnalzyer, который при обработке запроса наткнется на эти Identifier.
|
||||
/// Нам необходимо их пометить как имя базы данных и таблицы посколку по умолчанию стоит значение column
|
||||
dynamic_cast<ASTIdentifier &>(*args[1]).kind = ASTIdentifier::Database;
|
||||
dynamic_cast<ASTIdentifier &>(*args[2]).kind = ASTIdentifier::Table;
|
||||
|
||||
std::vector <std::vector< String> > names;
|
||||
std::vector<String> shards = parseDescription(descripton, 0, descripton.size(), ',');
|
||||
for (size_t i = 0; i < shards.size(); ++i)
|
||||
names.push_back(parseDescription(shards[i], 0, shards[i].size(), '|'));
|
||||
|
||||
cluster = new Cluster(context.getSettings(), context.getDataTypeFactory(), names);
|
||||
|
||||
return StorageDistributed::create(getName(), chooseColumns(*cluster, remote_database, remote_table, context), remote_database,
|
||||
remote_table, *cluster, context.getDataTypeFactory(), context.getSettings(), context);
|
||||
}
|
||||
|
||||
private:
|
||||
Poco::SharedPtr<Cluster> cluster; /// Ссылка на объект кластер передается в StorageDistributed и должен существовать до выполнения запроса
|
||||
|
||||
/// Узнать имена и типы столбцов для создания таблицы
|
||||
NamesAndTypesListPtr chooseColumns(Cluster & cluster, const String & database, const String &table, const Context & context) const
|
||||
{
|
||||
/// Запрос на описание таблицы
|
||||
String query = "DESC TABLE " + database + "." + table;
|
||||
Settings settings = context.getSettings();
|
||||
/// Отправляем на первый попавшийся сервер
|
||||
detail::ConnectionPoolEntry entry = (*cluster.pools.begin())->get(&settings);
|
||||
|
||||
NamesAndTypesList res;
|
||||
/// Парсим результат запроса и формируем NamesAndTypesList
|
||||
{
|
||||
BlockInputStreamPtr input = new RemoteBlockInputStream(entry, query, &settings, QueryProcessingStage::Complete);
|
||||
input->readPrefix();
|
||||
while (true)
|
||||
{
|
||||
Block current = input->read();
|
||||
if (!current)
|
||||
break;
|
||||
ColumnPtr name = current.getByName("name").column;
|
||||
ColumnPtr type = current.getByName("type").column;
|
||||
size_t size = name->size();
|
||||
for (size_t i = 0; i < size; ++i)
|
||||
{
|
||||
Field column, data_type;
|
||||
name->get(i, column);
|
||||
type->get(i, data_type);
|
||||
String column_name = column.get<String>();
|
||||
String data_type_name = data_type.get<String>();
|
||||
std::cerr << column_name << " " << data_type_name << std::endl;
|
||||
res.push_back(std::make_pair(column_name, context.getDataTypeFactory().get(data_type_name)));
|
||||
}
|
||||
}
|
||||
}
|
||||
return new NamesAndTypesList(res);
|
||||
}
|
||||
|
||||
/// Декартово произведение двух множеств строк, результат записываем на место первого аргумента
|
||||
void append(std::vector<String> & to, const std::vector<String> & what) const
|
||||
{
|
||||
if (what.empty()) return;
|
||||
if (to.empty())
|
||||
{
|
||||
to = what;
|
||||
return;
|
||||
}
|
||||
if (what.size() * to.size() > MAX_ADDRESSES)
|
||||
throw Exception("Storage Distributed, first argument generates too many result addresses",
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
std::vector<String> res;
|
||||
for (size_t i = 0; i < to.size(); ++i)
|
||||
for (size_t j = 0; j < what.size(); ++j)
|
||||
res.push_back(to[i] + what[j]);
|
||||
to.swap(res);
|
||||
}
|
||||
|
||||
/// Парсим число из подстроки
|
||||
bool parseId(const String & description, size_t l, size_t r, size_t & res) const
|
||||
{
|
||||
res = 0;
|
||||
for (size_t pos = l; pos < r; pos ++)
|
||||
{
|
||||
if (!isdigit(description[pos]))
|
||||
return false;
|
||||
res = res * 10 + description[pos] - '0';
|
||||
if (res > 1e15)
|
||||
return false;
|
||||
}
|
||||
return true;
|
||||
}
|
||||
|
||||
/* Парсит строку, генерирующую шарды и реплики. Spliter - один из двух символов | или '
|
||||
* в зависимости от того генерируются шарды или реплики.
|
||||
* Например:
|
||||
* host1,host2,... - порождает множество шардов из host1, host2, ...
|
||||
* host1|host2|... - порождает множество реплик из host1, host2, ...
|
||||
* abc{8..10}def - порождает множество шардов abc8def, abc9def, abc10def.
|
||||
* abc{08..10}def - порождает множество шардов abc08def, abc09def, abc10def.
|
||||
* abc{x,yy,z}def - порождает множество шардов abcxdef, abcyydef, abczdef.
|
||||
* abc{x|yy|z}def - порождает множество реплик abcxdef, abcyydef, abczdef.
|
||||
* abc{1..9}de{f,g,h} - прямое произведение, 27 шардов.
|
||||
* abc{1..9}de{0|1} - прямое произведение, 9 шардов, в каждом 2 реплики.
|
||||
*/
|
||||
std::vector<String> parseDescription(const String & description, size_t l, size_t r, char spliter) const
|
||||
{
|
||||
std::vector<String> res;
|
||||
std::vector<String> cur;
|
||||
|
||||
/// Пустая подстрока, означает множество из пустой строки
|
||||
if (l >= r)
|
||||
{
|
||||
res.push_back("");
|
||||
return res;
|
||||
}
|
||||
|
||||
for (size_t i = l; i < r; ++i)
|
||||
{
|
||||
/// Либо числовой интервал (8..10) либо аналогичное выражение в скобках
|
||||
if (description[i] == '{')
|
||||
{
|
||||
int cnt = 1;
|
||||
int last_dot = -1;
|
||||
size_t m;
|
||||
std::vector<String> buffer;
|
||||
bool have_spliter = false;
|
||||
|
||||
/// Ищем соответствующую нашей закрывающую скобку
|
||||
for (m = i + 1; m < r; ++m)
|
||||
{
|
||||
if (description[m] == '{') ++cnt;
|
||||
if (description[m] == '}') --cnt;
|
||||
if (description[m] == '.' && description[m-1] == '.') last_dot = m;
|
||||
if (description[m] == spliter) have_spliter = true;
|
||||
if (cnt == 0) break;
|
||||
}
|
||||
if (cnt != 0)
|
||||
throw Exception("Storage Distributed, incorrect brace sequence in first argument",
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
/// Наличие точки означает, что числовой интервал
|
||||
if (last_dot != -1)
|
||||
{
|
||||
size_t left, right;
|
||||
if (description[last_dot - 1] != '.')
|
||||
throw Exception("Storage Distributed, incorrect argument in braces (only one dot): " + description.substr(i, m - i + 1),
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
if (!parseId(description, i + 1, last_dot - 1, left))
|
||||
throw Exception("Storage Distributed, incorrect argument in braces (Incorrect left number): "
|
||||
+ description.substr(i, m - i + 1),
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
if (!parseId(description, last_dot + 1, m, right))
|
||||
throw Exception("Storage Distributed, incorrect argument in braces (Incorrect right number): "
|
||||
+ description.substr(i, m - i + 1),
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
if (left > right)
|
||||
throw Exception("Storage Distributed, incorrect argument in braces (left number is greater then right): "
|
||||
+ description.substr(i, m - i + 1),
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
if (right - left + 1 > MAX_ADDRESSES)
|
||||
throw Exception("Storage Distributed, first argument generates too many result addresses",
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
for (size_t id = left; id <= right; ++id)
|
||||
buffer.push_back(toString<uint64>(id));
|
||||
} else if (have_spliter) /// Если внутри есть текущий разделитель, то сгенерировать множество получаемых строк
|
||||
buffer = parseDescription(description, i + 1, m, spliter);
|
||||
else /// Иначе просто скопировать, порождение произойдет при вызове с правильным разделителем
|
||||
buffer.push_back(description.substr(i, m - i + 1));
|
||||
/// К текущему множеству строк добавить все возможные полученные продолжения
|
||||
append(cur, buffer);
|
||||
i = m;
|
||||
} else if (description[i] == spliter) {
|
||||
/// Если разделитель, то добавляем в ответ найденные строки
|
||||
res.insert(res.end(), cur.begin(), cur.end());
|
||||
cur.clear();
|
||||
} else {
|
||||
/// Иначе просто дописываем символ к текущим строкам
|
||||
std::vector<String> buffer;
|
||||
buffer.push_back(description.substr(i, 1));
|
||||
append(cur, buffer);
|
||||
}
|
||||
}
|
||||
res.insert(res.end(), cur.begin(), cur.end());
|
||||
if (res.size() > MAX_ADDRESSES)
|
||||
throw Exception("Storage Distributed, first argument generates too many result addresses",
|
||||
ErrorCodes::BAD_ARGUMENTS);
|
||||
return res;
|
||||
}
|
||||
|
||||
};
|
||||
|
||||
|
||||
}
|
||||
|
@ -125,6 +125,34 @@ Cluster::Cluster(const Settings & settings, const DataTypeFactory & data_type_fa
|
||||
throw Exception("No addresses listed in config", ErrorCodes::NO_ELEMENTS_IN_CONFIG);
|
||||
}
|
||||
|
||||
Cluster::Cluster(const Settings & settings, const DataTypeFactory & data_type_factory, std::vector< std::vector<String> > names):
|
||||
local_nodes_num(0)
|
||||
{
|
||||
for (size_t i = 0; i < names.size(); ++i)
|
||||
{
|
||||
Addresses current;
|
||||
for (size_t j = 0; j < names[i].size(); ++j)
|
||||
current.push_back(Address(Poco::Net::SocketAddress(names[i][j]), "default", ""));
|
||||
addresses_with_failover.push_back(current);
|
||||
}
|
||||
for (AddressesWithFailover::const_iterator it = addresses_with_failover.begin(); it != addresses_with_failover.end(); ++it)
|
||||
{
|
||||
ConnectionPools replicas;
|
||||
replicas.reserve(it->size());
|
||||
|
||||
for (Addresses::const_iterator jt = it->begin(); jt != it->end(); ++jt)
|
||||
{
|
||||
replicas.push_back(new ConnectionPool(
|
||||
settings.distributed_connections_pool_size,
|
||||
jt->host_port.host().toString(), jt->host_port.port(), "", jt->user, jt->password, data_type_factory, "server", Protocol::Compression::Enable,
|
||||
saturation(settings.connect_timeout_with_failover_ms, settings.limits.max_execution_time),
|
||||
saturation(settings.receive_timeout, settings.limits.max_execution_time),
|
||||
saturation(settings.send_timeout, settings.limits.max_execution_time)));
|
||||
}
|
||||
pools.push_back(new ConnectionPoolWithFailover(replicas, settings.load_balancing, settings.connections_with_failover_max_tries));
|
||||
}
|
||||
}
|
||||
|
||||
Poco::Timespan Cluster::saturation(const Poco::Timespan & v, const Poco::Timespan & limit)
|
||||
{
|
||||
if (limit.totalMicroseconds() == 0)
|
||||
|
@ -53,6 +53,21 @@ StoragePtr StorageDistributed::create(
|
||||
return (new StorageDistributed(name_, columns_, remote_database_, remote_table_, context_.getCluster(cluster_name), data_type_factory_, settings, context_, sign_column_name_))->thisPtr();
|
||||
}
|
||||
|
||||
|
||||
StoragePtr StorageDistributed::create(
|
||||
const std::string & name_,
|
||||
NamesAndTypesListPtr columns_,
|
||||
const String & remote_database_,
|
||||
const String & remote_table_,
|
||||
Cluster & cluster_,
|
||||
const DataTypeFactory & data_type_factory_,
|
||||
const Settings & settings,
|
||||
Context & context_,
|
||||
const String & sign_column_name_)
|
||||
{
|
||||
return (new StorageDistributed(name_, columns_, remote_database_, remote_table_, cluster_, data_type_factory_, settings, context_, sign_column_name_))->thisPtr();
|
||||
}
|
||||
|
||||
NameAndTypePair StorageDistributed::getColumn(const String &column_name) const
|
||||
{
|
||||
if (column_name == _host_column_name)
|
||||
|
@ -14,7 +14,6 @@ TableFunctionPtr TableFunctionFactory::get(
|
||||
const String & name,
|
||||
const Context & context) const
|
||||
{
|
||||
/// Немного неоптимально.
|
||||
if (name == "merge") return new TableFunctionMerge;
|
||||
else if (name == "remote") return new TableFunctionRemote;
|
||||
else
|
||||
|
Loading…
Reference in New Issue
Block a user