2018-04-09 13:52:39 +00:00
|
|
|
#include "getStructureOfRemoteTable.h"
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/Cluster.h>
|
|
|
|
#include <Interpreters/Context.h>
|
2019-01-09 12:21:04 +00:00
|
|
|
#include <Interpreters/ClusterProxy/executeQuery.h>
|
2018-02-15 18:54:12 +00:00
|
|
|
#include <Interpreters/InterpreterDescribeQuery.h>
|
2021-10-15 20:18:20 +00:00
|
|
|
#include <QueryPipeline/RemoteQueryExecutor.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <DataTypes/DataTypeFactory.h>
|
2019-02-19 17:02:51 +00:00
|
|
|
#include <DataTypes/DataTypeString.h>
|
|
|
|
#include <Columns/ColumnString.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/IStorage.h>
|
2018-03-12 13:47:01 +00:00
|
|
|
#include <Parsers/ExpressionListParsers.h>
|
|
|
|
#include <Parsers/parseQuery.h>
|
2018-07-24 14:05:37 +00:00
|
|
|
#include <Parsers/ASTFunction.h>
|
2019-10-08 18:42:22 +00:00
|
|
|
#include <Common/quoteString.h>
|
2020-03-31 01:51:03 +00:00
|
|
|
#include <Common/NetException.h>
|
2018-07-24 13:10:34 +00:00
|
|
|
#include <TableFunctions/TableFunctionFactory.h>
|
2016-05-13 03:22:16 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2019-10-11 14:44:55 +00:00
|
|
|
extern const int NO_REMOTE_SHARD_AVAILABLE;
|
2016-05-13 03:22:16 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2019-10-11 14:44:55 +00:00
|
|
|
ColumnsDescription getStructureOfRemoteTableInShard(
|
2020-10-02 22:28:46 +00:00
|
|
|
const Cluster & cluster,
|
2019-10-11 14:44:55 +00:00
|
|
|
const Cluster::ShardInfo & shard_info,
|
2020-03-31 01:14:32 +00:00
|
|
|
const StorageID & table_id,
|
2021-04-10 23:33:54 +00:00
|
|
|
ContextPtr context,
|
2019-10-11 14:44:55 +00:00
|
|
|
const ASTPtr & table_func_ptr)
|
|
|
|
{
|
2018-07-24 13:10:34 +00:00
|
|
|
String query;
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2018-07-24 13:10:34 +00:00
|
|
|
if (table_func_ptr)
|
|
|
|
{
|
|
|
|
if (shard_info.isLocal())
|
|
|
|
{
|
2020-10-14 12:19:29 +00:00
|
|
|
TableFunctionPtr table_function_ptr = TableFunctionFactory::instance().get(table_func_ptr, context);
|
|
|
|
return table_function_ptr->getActualTableStructure(context);
|
2018-07-24 13:10:34 +00:00
|
|
|
}
|
|
|
|
|
2018-07-25 12:31:47 +00:00
|
|
|
auto table_func_name = queryToString(table_func_ptr);
|
2018-07-24 13:10:34 +00:00
|
|
|
query = "DESC TABLE " + table_func_name;
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
if (shard_info.isLocal())
|
2020-06-17 16:39:58 +00:00
|
|
|
{
|
|
|
|
auto storage_ptr = DatabaseCatalog::instance().getTable(table_id, context);
|
|
|
|
return storage_ptr->getInMemoryMetadataPtr()->getColumns();
|
|
|
|
}
|
2018-07-24 13:10:34 +00:00
|
|
|
|
|
|
|
/// Request for a table description
|
2020-03-06 20:38:19 +00:00
|
|
|
query = "DESC TABLE " + table_id.getFullTableName();
|
2018-07-24 13:10:34 +00:00
|
|
|
}
|
2018-03-12 13:47:01 +00:00
|
|
|
|
|
|
|
ColumnsDescription res;
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
auto new_context = ClusterProxy::updateSettingsForCluster(cluster, context, context->getSettingsRef());
|
2019-02-19 17:02:51 +00:00
|
|
|
|
|
|
|
/// Expect only needed columns from the result of DESC TABLE. NOTE 'comment' column is ignored for compatibility reasons.
|
|
|
|
Block sample_block
|
|
|
|
{
|
|
|
|
{ ColumnString::create(), std::make_shared<DataTypeString>(), "name" },
|
|
|
|
{ ColumnString::create(), std::make_shared<DataTypeString>(), "type" },
|
|
|
|
{ ColumnString::create(), std::make_shared<DataTypeString>(), "default_type" },
|
|
|
|
{ ColumnString::create(), std::make_shared<DataTypeString>(), "default_expression" },
|
|
|
|
};
|
|
|
|
|
2019-01-09 12:21:04 +00:00
|
|
|
/// Execute remote query without restrictions (because it's not real user query, but part of implementation)
|
2021-10-08 14:03:54 +00:00
|
|
|
RemoteQueryExecutor executor(shard_info.pool, query, sample_block, new_context);
|
|
|
|
executor.setPoolMode(PoolMode::GET_ONE);
|
2018-07-24 13:10:34 +00:00
|
|
|
if (!table_func_ptr)
|
2021-10-08 14:03:54 +00:00
|
|
|
executor.setMainTable(table_id);
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
const DataTypeFactory & data_type_factory = DataTypeFactory::instance();
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2018-03-12 13:47:01 +00:00
|
|
|
ParserExpression expr_parser;
|
2018-07-25 12:31:47 +00:00
|
|
|
|
2021-10-08 14:03:54 +00:00
|
|
|
while (Block current = executor.read())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
ColumnPtr name = current.getByName("name").column;
|
|
|
|
ColumnPtr type = current.getByName("type").column;
|
2018-03-12 13:47:01 +00:00
|
|
|
ColumnPtr default_kind = current.getByName("default_type").column;
|
|
|
|
ColumnPtr default_expr = current.getByName("default_expression").column;
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t size = name->size();
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
for (size_t i = 0; i < size; ++i)
|
|
|
|
{
|
2019-03-14 15:20:51 +00:00
|
|
|
ColumnDescription column;
|
2018-03-12 13:47:01 +00:00
|
|
|
|
2019-03-14 15:20:51 +00:00
|
|
|
column.name = (*name)[i].get<const String &>();
|
2018-03-12 13:47:01 +00:00
|
|
|
|
2019-03-14 15:20:51 +00:00
|
|
|
String data_type_name = (*type)[i].get<const String &>();
|
|
|
|
column.type = data_type_factory.get(data_type_name);
|
2018-03-12 13:47:01 +00:00
|
|
|
|
2019-03-14 15:20:51 +00:00
|
|
|
String kind_name = (*default_kind)[i].get<const String &>();
|
|
|
|
if (!kind_name.empty())
|
|
|
|
{
|
|
|
|
column.default_desc.kind = columnDefaultKindFromString(kind_name);
|
2018-03-12 13:47:01 +00:00
|
|
|
String expr_str = (*default_expr)[i].get<const String &>();
|
2019-03-14 15:20:51 +00:00
|
|
|
column.default_desc.expression = parseQuery(
|
2021-04-10 23:33:54 +00:00
|
|
|
expr_parser, expr_str.data(), expr_str.data() + expr_str.size(), "default expression", 0, context->getSettingsRef().max_parser_depth);
|
2018-03-12 13:47:01 +00:00
|
|
|
}
|
2019-03-14 15:20:51 +00:00
|
|
|
|
|
|
|
res.add(column);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2021-10-08 14:03:54 +00:00
|
|
|
executor.finish();
|
2017-04-01 07:20:54 +00:00
|
|
|
return res;
|
2016-05-13 03:22:16 +00:00
|
|
|
}
|
|
|
|
|
2020-10-02 22:28:46 +00:00
|
|
|
ColumnsDescription getStructureOfRemoteTable(
|
|
|
|
const Cluster & cluster,
|
|
|
|
const StorageID & table_id,
|
2021-04-10 23:33:54 +00:00
|
|
|
ContextPtr context,
|
2020-10-02 22:28:46 +00:00
|
|
|
const ASTPtr & table_func_ptr)
|
|
|
|
{
|
|
|
|
const auto & shards_info = cluster.getShardsInfo();
|
|
|
|
|
|
|
|
std::string fail_messages;
|
|
|
|
|
|
|
|
for (const auto & shard_info : shards_info)
|
|
|
|
{
|
|
|
|
try
|
|
|
|
{
|
|
|
|
const auto & res = getStructureOfRemoteTableInShard(cluster, shard_info, table_id, context, table_func_ptr);
|
|
|
|
|
|
|
|
/// Expect at least some columns.
|
|
|
|
/// This is a hack to handle the empty block case returned by Connection when skip_unavailable_shards is set.
|
|
|
|
if (res.empty())
|
|
|
|
continue;
|
|
|
|
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
catch (const NetException &)
|
|
|
|
{
|
|
|
|
std::string fail_message = getCurrentExceptionMessage(false);
|
|
|
|
fail_messages += fail_message + '\n';
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
throw NetException(
|
|
|
|
"All attempts to get table structure failed. Log: \n\n" + fail_messages + "\n",
|
|
|
|
ErrorCodes::NO_REMOTE_SHARD_AVAILABLE);
|
|
|
|
}
|
|
|
|
|
2016-05-13 03:22:16 +00:00
|
|
|
}
|