ClickHouse/dbms/programs/copier/Internals.h

185 lines
5.2 KiB
C++
Raw Normal View History

2020-02-19 15:01:08 +00:00
#pragma once
#include <chrono>
#include <optional>
#include <Poco/Util/XMLConfiguration.h>
#include <Poco/Logger.h>
#include <Poco/ConsoleChannel.h>
#include <Poco/FormattingChannel.h>
#include <Poco/PatternFormatter.h>
#include <Poco/UUIDGenerator.h>
#include <Poco/File.h>
#include <Poco/Process.h>
#include <Poco/FileChannel.h>
#include <Poco/SplitterChannel.h>
#include <Poco/Util/HelpFormatter.h>
#include <boost/algorithm/string.hpp>
#include <common/logger_useful.h>
#include <Common/ThreadPool.h>
#include <Common/Exception.h>
#include <Common/ZooKeeper/ZooKeeper.h>
#include <Common/ZooKeeper/KeeperException.h>
#include <Common/getFQDNOrHostName.h>
#include <Common/isLocalAddress.h>
#include <Common/typeid_cast.h>
#include <Common/ClickHouseRevision.h>
#include <Common/formatReadable.h>
#include <Common/DNSResolver.h>
#include <Common/CurrentThread.h>
#include <Common/escapeForFileName.h>
#include <Common/getNumberOfPhysicalCPUCores.h>
#include <Common/ThreadStatus.h>
#include <Client/Connection.h>
#include <Interpreters/Context.h>
#include <Interpreters/InterpreterFactory.h>
#include <Interpreters/InterpreterExistsQuery.h>
#include <Interpreters/InterpreterShowCreateQuery.h>
#include <Interpreters/InterpreterDropQuery.h>
#include <Interpreters/InterpreterCreateQuery.h>
#include <Columns/ColumnString.h>
#include <Columns/ColumnsNumber.h>
#include <DataTypes/DataTypeString.h>
#include <Parsers/ParserCreateQuery.h>
#include <Parsers/parseQuery.h>
#include <Parsers/ParserQuery.h>
#include <Parsers/ASTCreateQuery.h>
#include <Parsers/queryToString.h>
#include <Parsers/ASTDropQuery.h>
#include <Parsers/ASTLiteral.h>
#include <Parsers/ASTExpressionList.h>
#include <Formats/FormatSettings.h>
#include <DataStreams/RemoteBlockInputStream.h>
#include <DataStreams/SquashingBlockInputStream.h>
#include <DataStreams/AsynchronousBlockInputStream.h>
#include <DataStreams/copyData.h>
#include <DataStreams/NullBlockOutputStream.h>
#include <IO/ConnectionTimeouts.h>
#include <IO/Operators.h>
#include <IO/ReadBufferFromString.h>
#include <IO/ReadBufferFromFile.h>
#include <Functions/registerFunctions.h>
#include <TableFunctions/registerTableFunctions.h>
#include <AggregateFunctions/registerAggregateFunctions.h>
#include <Storages/registerStorages.h>
#include <Storages/StorageDistributed.h>
#include <Dictionaries/registerDictionaries.h>
#include <Disks/registerDisks.h>
#include <Databases/DatabaseMemory.h>
#include <Common/StatusFile.h>
#include "Aliases.h"
2020-02-19 15:59:47 +00:00
namespace DB
{
2020-02-19 15:01:08 +00:00
namespace ErrorCodes
{
extern const int NO_ZOOKEEPER;
extern const int BAD_ARGUMENTS;
extern const int UNKNOWN_TABLE;
extern const int UNFINISHED;
extern const int UNKNOWN_ELEMENT_IN_CONFIG;
}
2020-02-19 15:45:49 +00:00
ConfigurationPtr getConfigurationFromXMLString(const std::string & xml_data);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
String getQuotedTable(const String & database, const String & table);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
String getQuotedTable(const DatabaseAndTableName & db_and_table);
2020-02-19 15:01:08 +00:00
2020-02-19 15:59:47 +00:00
enum class TaskState
{
2020-02-19 15:01:08 +00:00
Started = 0,
Finished,
Unknown
};
/// Used to mark status of shard partition tasks
2020-02-19 15:59:47 +00:00
struct TaskStateWithOwner
{
2020-02-19 15:01:08 +00:00
TaskStateWithOwner() = default;
2020-02-19 15:45:49 +00:00
TaskStateWithOwner(TaskState state_, const String & owner_) : state(state_), owner(owner_) {}
2020-02-19 15:01:08 +00:00
TaskState state{TaskState::Unknown};
String owner;
2020-02-20 09:06:00 +00:00
static String getData(TaskState state, const String &owner)
2020-02-19 20:50:27 +00:00
{
2020-02-19 15:01:08 +00:00
return TaskStateWithOwner(state, owner).toString();
}
2020-02-20 09:06:00 +00:00
String toString()
2020-02-19 20:50:27 +00:00
{
2020-02-19 15:01:08 +00:00
WriteBufferFromOwnString wb;
wb << static_cast<UInt32>(state) << "\n" << escape << owner;
return wb.str();
}
2020-02-20 09:06:00 +00:00
static TaskStateWithOwner fromString(const String & data)
2020-02-19 20:50:27 +00:00
{
2020-02-19 15:01:08 +00:00
ReadBufferFromString rb(data);
TaskStateWithOwner res;
UInt32 state;
rb >> state >> "\n" >> escape >> res.owner;
if (state >= static_cast<int>(TaskState::Unknown))
throw Exception("Unknown state " + data, ErrorCodes::LOGICAL_ERROR);
res.state = static_cast<TaskState>(state);
return res;
}
};
struct ShardPriority
{
UInt8 is_remote = 1;
size_t hostname_difference = 0;
UInt8 random = 0;
2020-02-20 09:06:00 +00:00
static bool greaterPriority(const ShardPriority & current, const ShardPriority & other)
2020-02-19 20:50:27 +00:00
{
2020-02-19 15:01:08 +00:00
return std::forward_as_tuple(current.is_remote, current.hostname_difference, current.random)
< std::forward_as_tuple(other.is_remote, other.hostname_difference, other.random);
}
};
/// Execution status of a task
enum class PartitionTaskStatus
{
Active,
Finished,
Error,
};
2020-02-19 15:59:47 +00:00
struct MultiTransactionInfo
{
2020-02-19 15:01:08 +00:00
int32_t code;
Coordination::Requests requests;
Coordination::Responses responses;
};
// Creates AST representing 'ENGINE = Distributed(cluster, db, table, [sharding_key])
2020-02-19 15:45:49 +00:00
std::shared_ptr<ASTStorage> createASTStorageDistributed(
const String & cluster_name, const String & database, const String & table,
const ASTPtr & sharding_key_ast = nullptr);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
BlockInputStreamPtr squashStreamIntoOneBlock(const BlockInputStreamPtr & stream);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
Block getBlockWithAllStreamData(const BlockInputStreamPtr & stream);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
bool isExtendedDefinitionStorage(const ASTPtr & storage_ast);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
ASTPtr extractPartitionKey(const ASTPtr & storage_ast);
2020-02-19 15:01:08 +00:00
2020-02-19 15:45:49 +00:00
ShardPriority getReplicasPriority(const Cluster::Addresses & replicas, const std::string & local_hostname, UInt8 random);
2020-02-19 15:01:08 +00:00
}