Merged with master

This commit is contained in:
Nikolai Kochetov 2018-09-06 21:30:03 +03:00
commit 8593145794
215 changed files with 1509 additions and 518 deletions

View File

@ -2,10 +2,10 @@
set(VERSION_REVISION 54407 CACHE STRING "")
set(VERSION_MAJOR 18 CACHE STRING "")
set(VERSION_MINOR 12 CACHE STRING "")
set(VERSION_PATCH 5 CACHE STRING "")
set(VERSION_GITHASH d8c528ea3973dbcfb68227fc0eff0feffa399d3d CACHE STRING "")
set(VERSION_DESCRIBE v18.12.5-testing CACHE STRING "")
set(VERSION_STRING 18.12.5 CACHE STRING "")
set(VERSION_PATCH 6 CACHE STRING "")
set(VERSION_GITHASH 79f33db9cc0f1cc6fde22be2cc88f43a4b167e18 CACHE STRING "")
set(VERSION_DESCRIBE v18.12.6-testing CACHE STRING "")
set(VERSION_STRING 18.12.6 CACHE STRING "")
# end of autochange
set(VERSION_EXTRA "" CACHE STRING "")

View File

@ -26,6 +26,7 @@
#include <IO/WriteHelpers.h>
#include <IO/Operators.h>
#include <IO/ConnectionTimeouts.h>
#include <IO/UseSSL.h>
#include <DataStreams/RemoteBlockInputStream.h>
#include <Interpreters/Context.h>
#include <Client/Connection.h>
@ -459,6 +460,8 @@ int mainEntryClickHouseBenchmark(int argc, char ** argv)
APPLY_FOR_SETTINGS(EXTRACT_SETTING)
#undef EXTRACT_SETTING
UseSSL use_ssl;
Benchmark benchmark(
options["concurrency"].as<unsigned>(),
options["delay"].as<double>(),

View File

@ -1,5 +1,5 @@
add_library (clickhouse-client-lib Client.cpp)
target_link_libraries (clickhouse-client-lib clickhouse_functions clickhouse_aggregate_functions ${LINE_EDITING_LIBS} ${Boost_PROGRAM_OPTIONS_LIBRARY})
target_link_libraries (clickhouse-client-lib clickhouse_common_io clickhouse_functions clickhouse_aggregate_functions ${LINE_EDITING_LIBS} ${Boost_PROGRAM_OPTIONS_LIBRARY})
if (READLINE_INCLUDE_DIR)
target_include_directories (clickhouse-client-lib SYSTEM PRIVATE ${READLINE_INCLUDE_DIR})
endif ()

View File

@ -41,6 +41,7 @@
#include <IO/ReadBufferFromString.h>
#include <IO/ReadHelpers.h>
#include <IO/WriteHelpers.h>
#include <IO/UseSSL.h>
#include <DataStreams/AsynchronousBlockInputStream.h>
#include <DataStreams/InternalTextLogsRowOutputStream.h>
#include <Parsers/ParserQuery.h>
@ -295,6 +296,8 @@ private:
int mainImpl()
{
UseSSL use_ssl;
registerFunctions();
registerAggregateFunctions();

View File

@ -1,5 +1,5 @@
add_library (clickhouse-local-lib LocalServer.cpp)
target_link_libraries (clickhouse-local-lib clickhouse-server-lib clickhouse_functions clickhouse_aggregate_functions clickhouse_table_functions ${Boost_PROGRAM_OPTIONS_LIBRARY})
target_link_libraries (clickhouse-local-lib clickhouse_common_io clickhouse-server-lib clickhouse_functions clickhouse_aggregate_functions clickhouse_table_functions ${Boost_PROGRAM_OPTIONS_LIBRARY})
if (CLICKHOUSE_SPLIT_BINARY)
add_executable (clickhouse-local clickhouse-local.cpp)

View File

@ -21,6 +21,7 @@
#include <IO/ReadBufferFromString.h>
#include <IO/WriteBufferFromString.h>
#include <IO/WriteBufferFromFileDescriptor.h>
#include <IO/UseSSL.h>
#include <Parsers/parseQuery.h>
#include <Parsers/IAST.h>
#include <common/ErrorHandlers.h>
@ -101,6 +102,8 @@ try
{
Logger * log = &logger();
UseSSL use_ssl;
if (!config().has("query") && !config().has("table-structure")) /// Nothing to process
{
if (config().hasOption("verbose"))

View File

@ -21,6 +21,7 @@
#include <IO/ReadHelpers.h>
#include <IO/WriteBufferFromFile.h>
#include <IO/ConnectionTimeouts.h>
#include <IO/UseSSL.h>
#include <Interpreters/Settings.h>
#include <common/ThreadPool.h>
#include <common/getMemoryAmount.h>
@ -1489,6 +1490,8 @@ try
auto timeouts = DB::ConnectionTimeouts::getTCPTimeoutsWithoutFailover(DB::Settings());
DB::UseSSL use_ssl;
DB::PerformanceTest performanceTest(options["host"].as<String>(),
options["port"].as<UInt16>(),
options["database"].as<String>(),

View File

@ -24,6 +24,7 @@
#include <Common/getNumberOfPhysicalCPUCores.h>
#include <Common/TaskStatsInfoGetter.h>
#include <IO/HTTPCommon.h>
#include <IO/UseSSL.h>
#include <Interpreters/AsynchronousMetrics.h>
#include <Interpreters/DDLWorker.h>
#include <Interpreters/ProcessList.h>
@ -94,6 +95,8 @@ int Server::main(const std::vector<std::string> & /*args*/)
{
Logger * log = &logger();
UseSSL use_ssl;
registerFunctions();
registerAggregateFunctions();
registerTableFunctions();
@ -319,6 +322,10 @@ int Server::main(const std::vector<std::string> & /*args*/)
if (mark_cache_size)
global_context->setMarkCache(mark_cache_size);
size_t compiled_expression_cache_size = config().getUInt64("compiled_expression_cache_size", std::numeric_limits<UInt64>::max());
if (compiled_expression_cache_size)
global_context->setCompiledExpressionCache(compiled_expression_cache_size);
/// Set path for format schema files
auto format_schema_path = Poco::File(config().getString("format_schema_path", path + "format_schemas/"));
global_context->setFormatSchemaPath(format_schema_path.path() + "/");
@ -473,7 +480,6 @@ int Server::main(const std::vector<std::string> & /*args*/)
if (config().has("https_port"))
{
#if USE_POCO_NETSSL
initSSL();
Poco::Net::SecureServerSocket socket;
auto address = socket_bind_listen(socket, listen_host, config().getInt("https_port"), /* secure = */ true);
socket.setReceiveTimeout(settings.http_receive_timeout);
@ -511,7 +517,6 @@ int Server::main(const std::vector<std::string> & /*args*/)
if (config().has("tcp_port_secure"))
{
#if USE_POCO_NETSSL
initSSL();
Poco::Net::SecureServerSocket socket;
auto address = socket_bind_listen(socket, listen_host, config().getInt("tcp_port_secure"), /* secure = */ true);
socket.setReceiveTimeout(settings.receive_timeout);
@ -551,7 +556,6 @@ int Server::main(const std::vector<std::string> & /*args*/)
if (config().has("interserver_https_port"))
{
#if USE_POCO_NETSSL
initSSL();
Poco::Net::SecureServerSocket socket;
auto address = socket_bind_listen(socket, listen_host, config().getInt("interserver_https_port"), /* secure = */ true);
socket.setReceiveTimeout(settings.http_receive_timeout);

View File

@ -5,5 +5,5 @@ add_headers_and_sources(clickhouse_common_config .)
add_library(clickhouse_common_config ${LINK_MODE} ${clickhouse_common_config_headers} ${clickhouse_common_config_sources})
target_link_libraries (clickhouse_common_config clickhouse_common_zookeeper string_utils)
target_link_libraries (clickhouse_common_config clickhouse_common_zookeeper string_utils ${Poco_XML_LIBRARY} ${Poco_Util_LIBRARY})
target_include_directories (clickhouse_common_config PRIVATE ${DBMS_INCLUDE_DIR})

View File

@ -115,14 +115,18 @@ public:
/// Insert the new value only if the token is still in present in insert_tokens.
/// (The token may be absent because of a concurrent reset() call).
bool result = false;
auto token_it = insert_tokens.find(key);
if (token_it != insert_tokens.end() && token_it->second.get() == token)
{
setImpl(key, token->value, cache_lock);
result = true;
}
if (!token->cleaned_up)
token_holder.cleanup(token_lock, cache_lock);
return std::make_pair(token->value, true);
return std::make_pair(token->value, result);
}
void getStats(size_t & out_hits, size_t & out_misses) const
@ -157,6 +161,29 @@ public:
virtual ~LRUCache() {}
protected:
using LRUQueue = std::list<Key>;
using LRUQueueIterator = typename LRUQueue::iterator;
struct Cell
{
bool expired(const Timestamp & last_timestamp, const Delay & delay) const
{
return (delay == Delay::zero()) ||
((last_timestamp > timestamp) && ((last_timestamp - timestamp) > delay));
}
MappedPtr value;
size_t size;
LRUQueueIterator queue_iterator;
Timestamp timestamp;
};
using Cells = std::unordered_map<Key, Cell, HashFunction>;
Cells cells;
mutable std::mutex mutex;
private:
/// Represents pending insertion attempt.
@ -222,36 +249,16 @@ private:
friend struct InsertTokenHolder;
using LRUQueue = std::list<Key>;
using LRUQueueIterator = typename LRUQueue::iterator;
struct Cell
{
bool expired(const Timestamp & last_timestamp, const Delay & delay) const
{
return (delay == Delay::zero()) ||
((last_timestamp > timestamp) && ((last_timestamp - timestamp) > delay));
}
MappedPtr value;
size_t size;
LRUQueueIterator queue_iterator;
Timestamp timestamp;
};
using Cells = std::unordered_map<Key, Cell, HashFunction>;
InsertTokenById insert_tokens;
LRUQueue queue;
Cells cells;
/// Total weight of values.
size_t current_size = 0;
const size_t max_size;
const Delay expiration_delay;
mutable std::mutex mutex;
std::atomic<size_t> hits {0};
std::atomic<size_t> misses {0};

View File

@ -89,6 +89,8 @@
M(CompileSuccess, "Number of times a compilation of generated C++ code was successful.") \
\
M(CompileFunction, "Number of times a compilation of generated LLVM code (to create fused function for complex expressions) was initiated.") \
M(CompiledFunctionExecute, "Number of times a compiled function was executed.") \
M(CompileExpressionsMicroseconds, "Total time spent for compilation of expressions to LLVM code.") \
\
M(ExternalSortWritePart, "") \
M(ExternalSortMerge, "") \
@ -168,7 +170,6 @@
M(OSReadChars, "Number of bytes read from filesystem, including page cache.") \
M(OSWriteChars, "Number of bytes written to filesystem, including page cache.") \
namespace ProfileEvents
{

View File

@ -15,6 +15,7 @@
#include <common/Types.h>
#include <common/unaligned.h>
#include <string>
#include <type_traits>
#define ROTL(x, b) static_cast<UInt64>(((x) << (b)) | ((x) >> (64 - (b))))
@ -139,6 +140,11 @@ public:
update(reinterpret_cast<const char *>(&x), sizeof(x));
}
void update(const std::string & x)
{
update(x.data(), x.length());
}
/// Get the result in some form. This can only be done once!
void get128(char * out)

View File

@ -20,14 +20,14 @@ namespace ErrorCodes
}
///
inline bool allowDecimalComparison(const IDataType & left_type, const IDataType & right_type)
inline bool allowDecimalComparison(const IDataType * left_type, const IDataType * right_type)
{
if (isDecimal(left_type))
{
if (isDecimal(right_type) || notDecimalButComparableToDecimal(right_type))
if (isDecimal(right_type) || isNotDecimalButComparableToDecimal(right_type))
return true;
}
else if (notDecimalButComparableToDecimal(left_type) && isDecimal(right_type))
else if (isNotDecimalButComparableToDecimal(left_type) && isDecimal(right_type))
return true;
return false;
}

View File

@ -36,7 +36,7 @@ try
Context context = Context::createGlobal();
ExpressionAnalyzer analyzer(ast, context, {}, {NameAndTypePair("number", std::make_shared<DataTypeUInt64>())});
ExpressionActionsChain chain;
ExpressionActionsChain chain(context);
analyzer.appendSelect(chain, false);
analyzer.appendProjectResult(chain);
chain.finalize();

View File

@ -41,7 +41,7 @@ try
Context context = Context::createGlobal();
ExpressionAnalyzer analyzer(ast, context, {}, {NameAndTypePair("number", std::make_shared<DataTypeUInt64>())});
ExpressionActionsChain chain;
ExpressionActionsChain chain(context);
analyzer.appendSelect(chain, false);
analyzer.appendProjectResult(chain);
chain.finalize();

View File

@ -134,20 +134,12 @@ public:
bool textCanContainOnlyValidUTF8() const override { return true; }
bool isComparable() const override { return true; }
bool isValueRepresentedByNumber() const override { return true; }
bool isValueRepresentedByInteger() const override { return true; }
bool isValueRepresentedByUnsignedInteger() const override { return false; }
bool isValueUnambiguouslyRepresentedInContiguousMemoryRegion() const override { return true; }
bool haveMaximumSizeOfValue() const override { return true; }
size_t getSizeOfValueInMemory() const override { return sizeof(T); }
bool isCategorial() const override { return isValueRepresentedByInteger(); }
bool canBeUsedAsVersion() const override { return false; }
bool isSummable() const override { return true; }
bool canBeUsedInBitOperations() const override { return false; }
bool isUnsignedInteger() const override { return false; }
bool canBeUsedInBooleanContext() const override { return true; }
bool isNumber() const override { return true; }
bool isInteger() const override { return false; }
bool canBeInsideNullable() const override { return true; }
/// Decimal specific
@ -255,17 +247,6 @@ inline const DataTypeDecimal<T> * checkDecimal(const IDataType & data_type)
return typeid_cast<const DataTypeDecimal<T> *>(&data_type);
}
inline bool isDecimal(const IDataType & data_type)
{
if (typeid_cast<const DataTypeDecimal<Decimal32> *>(&data_type))
return true;
if (typeid_cast<const DataTypeDecimal<Decimal64> *>(&data_type))
return true;
if (typeid_cast<const DataTypeDecimal<Decimal128> *>(&data_type))
return true;
return false;
}
inline UInt32 getDecimalScale(const IDataType & data_type)
{
if (auto * decimal_type = checkDecimal<Decimal32>(data_type))
@ -278,20 +259,6 @@ inline UInt32 getDecimalScale(const IDataType & data_type)
}
///
inline bool notDecimalButComparableToDecimal(const IDataType & data_type)
{
if (data_type.isInteger())
return true;
return false;
}
///
inline bool comparableToDecimal(const IDataType & data_type)
{
if (data_type.isInteger())
return true;
return isDecimal(data_type);
}
template <typename DataType> constexpr bool IsDecimal = false;
template <> constexpr bool IsDecimal<DataTypeDecimal<Decimal32>> = true;

View File

@ -423,5 +423,77 @@ public:
};
struct DataTypeExtractor
{
TypeIndex idx;
DataTypeExtractor(const IDataType * data_type)
: idx(data_type->getTypeId())
{}
bool isUInt8() const { return idx == TypeIndex::UInt8; }
bool isUInt16() const { return idx == TypeIndex::UInt16; }
bool isUInt32() const { return idx == TypeIndex::UInt32; }
bool isUInt64() const { return idx == TypeIndex::UInt64; }
bool isUInt128() const { return idx == TypeIndex::UInt128; }
bool isUInt() const { return isUInt8() || isUInt16() || isUInt32() || isUInt64() || isUInt128(); }
bool isInt8() const { return idx == TypeIndex::Int8; }
bool isInt16() const { return idx == TypeIndex::Int16; }
bool isInt32() const { return idx == TypeIndex::Int32; }
bool isInt64() const { return idx == TypeIndex::Int64; }
bool isInt128() const { return idx == TypeIndex::Int128; }
bool isInt() const { return isInt8() || isInt16() || isInt32() || isInt64() || isInt128(); }
bool isDecimal32() const { return idx == TypeIndex::Decimal32; }
bool isDecimal64() const { return idx == TypeIndex::Decimal64; }
bool isDecimal128() const { return idx == TypeIndex::Decimal128; }
bool isDecimal() const { return isDecimal32() || isDecimal64() || isDecimal128(); }
bool isFloat32() const { return idx == TypeIndex::Float32; }
bool isFloat64() const { return idx == TypeIndex::Float64; }
bool isFloat() const { return isFloat32() || isFloat64(); }
bool isEnum8() const { return idx == TypeIndex::Enum8; }
bool isEnum16() const { return idx == TypeIndex::Enum16; }
bool isEnum() const { return isEnum8() || isEnum16(); }
bool isDate() const { return idx == TypeIndex::Date; }
bool isDateTime() const { return idx == TypeIndex::DateTime; }
bool isDateOrDateTime() const { return isDate() || isDateTime(); }
bool isString() const { return idx == TypeIndex::String; }
bool isFixedString() const { return idx == TypeIndex::FixedString; }
bool isStringOrFixedString() const { return isString() || isFixedString(); }
bool isUUID() const { return idx == TypeIndex::UUID; }
bool isArray() const { return idx == TypeIndex::Array; }
bool isTuple() const { return idx == TypeIndex::Tuple; }
};
/// IDataType helpers (alternative for IDataType virtual methods)
inline bool isEnum(const IDataType * data_type)
{
return DataTypeExtractor(data_type).isEnum();
}
inline bool isDecimal(const IDataType * data_type)
{
return DataTypeExtractor(data_type).isDecimal();
}
inline bool isNotDecimalButComparableToDecimal(const IDataType * data_type)
{
DataTypeExtractor which(data_type);
return which.isInt() || which.isUInt();
}
inline bool isCompilableType(const IDataType * data_type)
{
return data_type->isValueRepresentedByNumber() && !isDecimal(data_type);
}
}

View File

@ -1,3 +1,5 @@
#include <unordered_set>
#include <IO/WriteBufferFromString.h>
#include <IO/Operators.h>
#include <Common/typeid_cast.h>
@ -11,6 +13,7 @@
#include <DataTypes/DataTypeString.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypesNumber.h>
#include <DataTypes/DataTypesDecimal.h>
namespace DB
@ -185,22 +188,19 @@ DataTypePtr getLeastSupertype(const DataTypes & types)
/// Non-recursive rules
std::unordered_set<TypeIndex> type_ids;
for (const auto & type : types)
type_ids.insert(type->getTypeId());
/// For String and FixedString, or for different FixedStrings, the common type is String.
/// No other types are compatible with Strings. TODO Enums?
{
bool have_string = false;
bool all_strings = true;
UInt32 have_string = type_ids.count(TypeIndex::String);
UInt32 have_fixed_string = type_ids.count(TypeIndex::FixedString);
for (const auto & type : types)
{
if (type->isStringOrFixedString())
have_string = true;
else
all_strings = false;
}
if (have_string)
if (have_string || have_fixed_string)
{
bool all_strings = type_ids.size() == (have_string + have_fixed_string);
if (!all_strings)
throw Exception(getExceptionMessagePrefix(types) + " because some of them are String/FixedString and some of them are not", ErrorCodes::NO_COMMON_TYPE);
@ -210,19 +210,12 @@ DataTypePtr getLeastSupertype(const DataTypes & types)
/// For Date and DateTime, the common type is DateTime. No other types are compatible.
{
bool have_date_or_datetime = false;
bool all_date_or_datetime = true;
UInt32 have_date = type_ids.count(TypeIndex::Date);
UInt32 have_datetime = type_ids.count(TypeIndex::DateTime);
for (const auto & type : types)
{
if (type->isDateOrDateTime())
have_date_or_datetime = true;
else
all_date_or_datetime = false;
}
if (have_date_or_datetime)
if (have_date || have_datetime)
{
bool all_date_or_datetime = type_ids.size() == (have_date + have_datetime);
if (!all_date_or_datetime)
throw Exception(getExceptionMessagePrefix(types) + " because some of them are Date/DateTime and some of them are not", ErrorCodes::NO_COMMON_TYPE);
@ -230,6 +223,35 @@ DataTypePtr getLeastSupertype(const DataTypes & types)
}
}
/// Decimals
{
UInt32 have_decimal32 = type_ids.count(TypeIndex::Decimal32);
UInt32 have_decimal64 = type_ids.count(TypeIndex::Decimal64);
UInt32 have_decimal128 = type_ids.count(TypeIndex::Decimal128);
if (have_decimal32 || have_decimal64 || have_decimal128)
{
bool all_are_decimals = type_ids.size() == (have_decimal32 + have_decimal64 + have_decimal128);
if (!all_are_decimals)
throw Exception(getExceptionMessagePrefix(types) + " because some of them are Decimals and some are not",
ErrorCodes::NO_COMMON_TYPE);
UInt32 max_scale = 0;
for (const auto & type : types)
{
UInt32 scale = getDecimalScale(*type);
if (scale > max_scale)
max_scale = scale;
}
if (have_decimal128)
return std::make_shared<DataTypeDecimal<Decimal128>>(DataTypeDecimal<Decimal128>::maxPrecision(), max_scale);
if (have_decimal64)
return std::make_shared<DataTypeDecimal<Decimal64>>(DataTypeDecimal<Decimal64>::maxPrecision(), max_scale);
return std::make_shared<DataTypeDecimal<Decimal32>>(DataTypeDecimal<Decimal32>::maxPrecision(), max_scale);
}
}
/// For numeric types, the most complicated part.
{
bool all_numbers = true;

View File

@ -165,8 +165,6 @@ DictionarySourcePtr DictionarySourceFactory::create(
if (dict_struct.has_expressions)
throw Exception{"Dictionary source of type `http` does not support attribute expressions", ErrorCodes::LOGICAL_ERROR};
// Used for https queries
initSSL();
return std::make_unique<HTTPDictionarySource>(dict_struct, config, config_prefix + ".http", sample_block, context);
}
else if ("library" == source_type)

View File

@ -733,6 +733,14 @@ struct ArrayIndexGenericNullImpl
}
};
inline bool allowArrayIndex(const IDataType * data_type0, const IDataType * data_type1)
{
return ((data_type0->isNumber() || isEnum(data_type0)) && data_type1->isNumber())
|| data_type0->equals(*data_type1);
}
template <typename IndexConv, typename Name>
class FunctionArrayIndex : public IFunction
{
@ -1010,9 +1018,7 @@ public:
DataTypePtr observed_type0 = removeNullable(array_type->getNestedType());
DataTypePtr observed_type1 = removeNullable(arguments[1]);
/// We also support arrays of Enum type (that are represented by number) to search numeric values.
if (!(observed_type0->isValueRepresentedByNumber() && observed_type1->isNumber())
&& !observed_type0->equals(*observed_type1))
if (!allowArrayIndex(observed_type0.get(), observed_type1.get()))
throw Exception("Types of array and 2nd argument of function "
+ getName() + " must be identical up to nullability or numeric types or Enum and numeric type. Passed: "
+ arguments[0]->getName() + " and " + arguments[1]->getName() + ".",

View File

@ -817,17 +817,9 @@ private:
const IColumn * column_number = left_is_num ? col_left_untyped : col_right_untyped;
const IDataType * number_type = left_is_num ? left_type.get() : right_type.get();
bool is_date = false;
bool is_date_time = false;
bool is_uuid = false;
bool is_enum8 = false;
bool is_enum16 = false;
DataTypeExtractor which(number_type);
const auto legal_types = (is_date = checkAndGetDataType<DataTypeDate>(number_type))
|| (is_date_time = checkAndGetDataType<DataTypeDateTime>(number_type))
|| (is_uuid = checkAndGetDataType<DataTypeUUID>(number_type))
|| (is_enum8 = checkAndGetDataType<DataTypeEnum8>(number_type))
|| (is_enum16 = checkAndGetDataType<DataTypeEnum16>(number_type));
const bool legal_types = which.isDateOrDateTime() || which.isEnum() || which.isUUID();
const auto column_string = checkAndGetColumnConst<ColumnString>(column_string_untyped);
if (!column_string || !legal_types)
@ -835,7 +827,7 @@ private:
StringRef string_value = column_string->getDataAt(0);
if (is_date)
if (which.isDate())
{
DayNum date;
ReadBufferFromMemory in(string_value.data, string_value.size);
@ -849,7 +841,7 @@ private:
left_is_num ? col_left_untyped : parsed_const_date,
left_is_num ? parsed_const_date : col_right_untyped);
}
else if (is_date_time)
else if (which.isDateTime())
{
time_t date_time;
ReadBufferFromMemory in(string_value.data, string_value.size);
@ -863,7 +855,7 @@ private:
left_is_num ? col_left_untyped : parsed_const_date_time,
left_is_num ? parsed_const_date_time : col_right_untyped);
}
else if (is_uuid)
else if (which.isUUID())
{
UUID uuid;
ReadBufferFromMemory in(string_value.data, string_value.size);
@ -878,10 +870,10 @@ private:
left_is_num ? parsed_const_uuid : col_right_untyped);
}
else if (is_enum8)
else if (which.isEnum8())
executeEnumWithConstString<DataTypeEnum8>(block, result, column_number, column_string,
number_type, left_is_num, input_rows_count);
else if (is_enum16)
else if (which.isEnum16())
executeEnumWithConstString<DataTypeEnum16>(block, result, column_number, column_string,
number_type, left_is_num, input_rows_count);
@ -1085,62 +1077,26 @@ public:
/// Get result types by argument types. If the function does not apply to these arguments, throw an exception.
DataTypePtr getReturnTypeImpl(const DataTypes & arguments) const override
{
bool left_is_date = false;
bool left_is_date_time = false;
bool left_is_uuid = false;
bool left_is_enum8 = false;
bool left_is_enum16 = false;
bool left_is_string = false;
bool left_is_fixed_string = false;
const DataTypeTuple * left_tuple = nullptr;
DataTypeExtractor left(arguments[0].get());
DataTypeExtractor right(arguments[1].get());
false
|| (left_is_date = checkAndGetDataType<DataTypeDate>(arguments[0].get()))
|| (left_is_date_time = checkAndGetDataType<DataTypeDateTime>(arguments[0].get()))
|| (left_is_enum8 = checkAndGetDataType<DataTypeEnum8>(arguments[0].get()))
|| (left_is_uuid = checkAndGetDataType<DataTypeUUID>(arguments[0].get()))
|| (left_is_enum16 = checkAndGetDataType<DataTypeEnum16>(arguments[0].get()))
|| (left_is_string = checkAndGetDataType<DataTypeString>(arguments[0].get()))
|| (left_is_fixed_string = checkAndGetDataType<DataTypeFixedString>(arguments[0].get()))
|| (left_tuple = checkAndGetDataType<DataTypeTuple>(arguments[0].get()));
const bool left_is_enum = left_is_enum8 || left_is_enum16;
bool right_is_date = false;
bool right_is_date_time = false;
bool right_is_uuid = false;
bool right_is_enum8 = false;
bool right_is_enum16 = false;
bool right_is_string = false;
bool right_is_fixed_string = false;
const DataTypeTuple * right_tuple = nullptr;
false
|| (right_is_date = checkAndGetDataType<DataTypeDate>(arguments[1].get()))
|| (right_is_date_time = checkAndGetDataType<DataTypeDateTime>(arguments[1].get()))
|| (right_is_uuid = checkAndGetDataType<DataTypeUUID>(arguments[1].get()))
|| (right_is_enum8 = checkAndGetDataType<DataTypeEnum8>(arguments[1].get()))
|| (right_is_enum16 = checkAndGetDataType<DataTypeEnum16>(arguments[1].get()))
|| (right_is_string = checkAndGetDataType<DataTypeString>(arguments[1].get()))
|| (right_is_fixed_string = checkAndGetDataType<DataTypeFixedString>(arguments[1].get()))
|| (right_tuple = checkAndGetDataType<DataTypeTuple>(arguments[1].get()));
const bool right_is_enum = right_is_enum8 || right_is_enum16;
const DataTypeTuple * left_tuple = checkAndGetDataType<DataTypeTuple>(arguments[0].get());
const DataTypeTuple * right_tuple = checkAndGetDataType<DataTypeTuple>(arguments[1].get());
if (!((arguments[0]->isValueRepresentedByNumber() && arguments[1]->isValueRepresentedByNumber())
|| ((left_is_string || left_is_fixed_string) && (right_is_string || right_is_fixed_string))
|| (left_is_date && right_is_date)
|| (left_is_date && right_is_string) /// You can compare the date, datetime and an enumeration with a constant string.
|| (left_is_string && right_is_date)
|| (left_is_date_time && right_is_date_time)
|| (left_is_date_time && right_is_string)
|| (left_is_string && right_is_date_time)
|| (left_is_uuid && right_is_uuid)
|| (left_is_uuid && right_is_string)
|| (left_is_string && right_is_uuid)
|| (left_is_enum && right_is_enum && arguments[0]->getName() == arguments[1]->getName()) /// only equivalent enum type values can be compared against
|| (left_is_enum && right_is_string)
|| (left_is_string && right_is_enum)
|| (left.isStringOrFixedString() && right.isStringOrFixedString())
|| (left.isDate() && right.isDate())
|| (left.isDate() && right.isString()) /// You can compare the date, datetime and an enumeration with a constant string.
|| (left.isString() && right.isDate())
|| (left.isDateTime() && right.isDateTime())
|| (left.isDateTime() && right.isString())
|| (left.isString() && right.isDateTime())
|| (left.isUUID() && right.isUUID())
|| (left.isUUID() && right.isString())
|| (left.isString() && right.isUUID())
|| (left.isEnum() && right.isEnum() && arguments[0]->getName() == arguments[1]->getName()) /// only equivalent enum type values can be compared against
|| (left.isEnum() && right.isString())
|| (left.isString() && right.isEnum())
|| (left_tuple && right_tuple && left_tuple->getElements().size() == right_tuple->getElements().size())
|| (arguments[0]->equals(*arguments[1]))))
{
@ -1203,9 +1159,9 @@ public:
{
executeTuple(block, result, col_with_type_and_name_left, col_with_type_and_name_right, input_rows_count);
}
else if (isDecimal(*left_type) || isDecimal(*right_type))
else if (isDecimal(left_type.get()) || isDecimal(right_type.get()))
{
if (!allowDecimalComparison(*left_type, *right_type))
if (!allowDecimalComparison(left_type.get(), right_type.get()))
throw Exception("No operation " + getName() + " between " + left_type->getName() + " and " + right_type->getName(),
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
@ -1237,7 +1193,7 @@ public:
auto isFloatingPoint = &typeIsEither<DataTypeFloat32, DataTypeFloat64>;
if ((isBigInteger(*types[0]) && isFloatingPoint(*types[1])) || (isBigInteger(*types[1]) && isFloatingPoint(*types[0])))
return false; /// TODO: implement (double, int_N where N > double's mantissa width)
return types[0]->isValueRepresentedByNumber() && types[1]->isValueRepresentedByNumber();
return isCompilableType(types[0].get()) && isCompilableType(types[1].get());
}
llvm::Value * compileImpl(llvm::IRBuilderBase & builder, const DataTypes & types, ValuePlaceholders values) const override

View File

@ -122,7 +122,7 @@ public:
bool isCompilableImpl(const DataTypes & types) const override
{
for (const auto & type : types)
if (!removeNullable(type)->isValueRepresentedByNumber())
if (!isCompilableType(removeNullable(type).get()))
return false;
return true;
}

View File

@ -242,6 +242,12 @@ public:
};
inline bool allowIntHash(const IDataType * data_type)
{
return data_type->isValueRepresentedByNumber();
}
template <typename Impl, typename Name>
class FunctionIntHash : public IFunction
{
@ -285,7 +291,7 @@ public:
DataTypePtr getReturnTypeImpl(const DataTypes & arguments) const override
{
if (!arguments[0]->isValueRepresentedByNumber())
if (!allowIntHash(arguments[0].get()))
throw Exception("Illegal type " + arguments[0]->getName() + " of argument of function " + getName(),
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);

View File

@ -39,24 +39,6 @@ void setResponseDefaultHeaders(Poco::Net::HTTPServerResponse & response, unsigne
response.set("Keep-Alive", "timeout=" + std::to_string(timeout.totalSeconds()));
}
void initSSL()
{
// http://stackoverflow.com/questions/18315472/https-request-in-c-using-poco
#if USE_POCO_NETSSL
struct Initializer
{
Initializer()
{
Poco::Net::initializeSSL();
}
};
static Initializer initializer;
#endif
}
std::unique_ptr<Poco::Net::HTTPClientSession> makeHTTPSession(const Poco::URI & uri, const ConnectionTimeouts & timeouts)
{
bool is_ssl = static_cast<bool>(uri.getScheme() == "https");

View File

@ -29,10 +29,6 @@ const int HTTP_TOO_MANY_REQUESTS = 429;
void setResponseDefaultHeaders(Poco::Net::HTTPServerResponse & response, unsigned keep_alive_timeout);
/// Call this method if you are going to make HTTPS requests. It's safe to call it many time from different threads.
void initSSL();
/// Create session object to perform requests and set required parameters.
std::unique_ptr<Poco::Net::HTTPClientSession> makeHTTPSession(const Poco::URI & uri, const ConnectionTimeouts & timeouts);

24
dbms/src/IO/UseSSL.cpp Normal file
View File

@ -0,0 +1,24 @@
#include "UseSSL.h"
#include <Common/config.h>
#if USE_POCO_NETSSL
#include <Poco/Net/SSLManager.h>
#endif
namespace DB
{
UseSSL::UseSSL()
{
#if USE_POCO_NETSSL
Poco::Net::initializeSSL();
#endif
}
UseSSL::~UseSSL()
{
#if USE_POCO_NETSSL
Poco::Net::uninitializeSSL();
#endif
}
}

13
dbms/src/IO/UseSSL.h Normal file
View File

@ -0,0 +1,13 @@
#pragma once
#include <boost/noncopyable.hpp>
namespace DB
{
// http://stackoverflow.com/questions/18315472/https-request-in-c-using-poco
struct UseSSL : private boost::noncopyable
{
UseSSL();
~UseSSL();
};
}

View File

@ -1,8 +1,10 @@
#include <Interpreters/AsynchronousMetrics.h>
#include <Interpreters/ExpressionJIT.h>
#include <Common/Exception.h>
#include <Common/setThreadName.h>
#include <Common/CurrentMetrics.h>
#include <Common/typeid_cast.h>
#include <Common/config.h>
#include <Storages/MarkCache.h>
#include <Storages/StorageMergeTree.h>
#include <Storages/StorageReplicatedMergeTree.h>
@ -132,6 +134,16 @@ void AsynchronousMetrics::update()
}
}
#if USE_EMBEDDED_COMPILER
{
if (auto compiled_expression_cache = context.getCompiledExpressionCache())
{
set("CompiledExpressionCacheBytes", compiled_expression_cache->weight());
set("CompiledExpressionCacheCount", compiled_expression_cache->count());
}
}
#endif
set("Uptime", context.getUptimeSeconds());
{

View File

@ -24,12 +24,14 @@
#include <TableFunctions/TableFunctionFactory.h>
#include <Interpreters/ActionLocksManager.h>
#include <Interpreters/Settings.h>
#include <Interpreters/ExpressionJIT.h>
#include <Interpreters/RuntimeComponentsFactory.h>
#include <Interpreters/ISecurityManager.h>
#include <Interpreters/Quota.h>
#include <Interpreters/EmbeddedDictionaries.h>
#include <Interpreters/ExternalDictionaries.h>
#include <Interpreters/ExternalModels.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/ProcessList.h>
#include <Interpreters/Cluster.h>
#include <Interpreters/InterserverIOHandler.h>
@ -54,6 +56,7 @@
namespace ProfileEvents
{
extern const Event ContextLock;
extern const Event CompiledCacheSizeBytes;
}
namespace CurrentMetrics
@ -174,6 +177,10 @@ struct ContextShared
ConfigurationPtr clusters_config; /// Soteres updated configs
mutable std::mutex clusters_mutex; /// Guards clusters and clusters_config
#if USE_EMBEDDED_COMPILER
std::shared_ptr<CompiledExpressionCache> compiled_expression_cache;
#endif
bool shutdown_called = false;
/// Do not allow simultaneous execution of DDL requests on the same table.
@ -1805,6 +1812,35 @@ Context::SampleBlockCache & Context::getSampleBlockCache() const
return getQueryContext().sample_block_cache;
}
#if USE_EMBEDDED_COMPILER
std::shared_ptr<CompiledExpressionCache> Context::getCompiledExpressionCache() const
{
auto lock = getLock();
return shared->compiled_expression_cache;
}
void Context::setCompiledExpressionCache(size_t cache_size)
{
auto lock = getLock();
if (shared->compiled_expression_cache)
throw Exception("Compiled expressions cache has been already created.", ErrorCodes::LOGICAL_ERROR);
shared->compiled_expression_cache = std::make_shared<CompiledExpressionCache>(cache_size);
}
void Context::dropCompiledExpressionCache() const
{
auto lock = getLock();
if (shared->compiled_expression_cache)
shared->compiled_expression_cache->reset();
}
#endif
std::shared_ptr<ActionLocksManager> Context::getActionLocksManager()
{
auto lock = getLock();

View File

@ -8,7 +8,9 @@
#include <thread>
#include <atomic>
#include <Common/config.h>
#include <common/MultiVersion.h>
#include <Common/LRUCache.h>
#include <Core/Types.h>
#include <Core/NamesAndTypes.h>
#include <Core/Block.h>
@ -77,6 +79,11 @@ using SystemLogsPtr = std::shared_ptr<SystemLogs>;
class ActionLocksManager;
using ActionLocksManagerPtr = std::shared_ptr<ActionLocksManager>;
#if USE_EMBEDDED_COMPILER
class CompiledExpressionCache;
#endif
/// (database name, table name)
using DatabaseAndTableName = std::pair<String, String>;
@ -432,6 +439,12 @@ public:
SampleBlockCache & getSampleBlockCache() const;
#if USE_EMBEDDED_COMPILER
std::shared_ptr<CompiledExpressionCache> getCompiledExpressionCache() const;
void setCompiledExpressionCache(size_t cache_size);
void dropCompiledExpressionCache() const;
#endif
private:
/** Check if the current client has access to the specified database.
* If access is denied, throw an exception.

View File

@ -1,5 +1,6 @@
#include <Common/config.h>
#include <Common/ProfileEvents.h>
#include <Common/SipHash.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/ExpressionJIT.h>
#include <Interpreters/Join.h>
@ -17,6 +18,7 @@
namespace ProfileEvents
{
extern const Event FunctionExecute;
extern const Event CompiledFunctionExecute;
}
namespace DB
@ -374,6 +376,8 @@ void ExpressionAction::execute(Block & block, std::unordered_map<std::string, si
block.insert({ nullptr, result_type, result_name});
ProfileEvents::increment(ProfileEvents::FunctionExecute);
if (is_function_compiled)
ProfileEvents::increment(ProfileEvents::CompiledFunctionExecute);
function->execute(block, arguments, num_columns_without_result, input_rows_count);
break;
@ -535,7 +539,7 @@ std::string ExpressionAction::toString() const
break;
case APPLY_FUNCTION:
ss << "FUNCTION " << result_name << " "
ss << "FUNCTION " << result_name << " " << (is_function_compiled ? "[compiled] " : "")
<< (result_type ? result_type->getName() : "(no type)") << " = "
<< (function ? function->getName() : "(no function)") << "(";
for (size_t i = 0; i < argument_names.size(); ++i)
@ -799,7 +803,7 @@ void ExpressionActions::finalize(const Names & output_columns)
/// This has to be done before removing redundant actions and inserting REMOVE_COLUMNs
/// because inlining may change dependency sets.
if (settings.compile_expressions)
compileFunctions(actions, output_columns, sample_block);
compileFunctions(actions, output_columns, sample_block, compilation_cache);
#endif
/// Which columns are needed to perform actions from the current to the last.
@ -1111,13 +1115,119 @@ BlockInputStreamPtr ExpressionActions::createStreamWithNonJoinedDataIfFullOrRigh
}
/// It is not important to calculate the hash of individual strings or their concatenation
size_t ExpressionAction::ActionHash::operator()(const ExpressionAction & action) const
{
SipHash hash;
hash.update(action.type);
hash.update(action.is_function_compiled);
switch(action.type)
{
case ADD_COLUMN:
hash.update(action.result_name);
if (action.result_type)
hash.update(action.result_type->getName());
if (action.added_column)
hash.update(action.added_column->getName());
break;
case REMOVE_COLUMN:
hash.update(action.source_name);
break;
case COPY_COLUMN:
hash.update(action.result_name);
hash.update(action.source_name);
break;
case APPLY_FUNCTION:
hash.update(action.result_name);
if (action.result_type)
hash.update(action.result_type->getName());
if (action.function)
{
hash.update(action.function->getName());
for (const auto & arg_type : action.function->getArgumentTypes())
hash.update(arg_type->getName());
}
for (const auto & arg_name : action.argument_names)
hash.update(arg_name);
break;
case ARRAY_JOIN:
hash.update(action.array_join_is_left);
for (const auto & col : action.array_joined_columns)
hash.update(col);
break;
case JOIN:
for (const auto & col : action.columns_added_by_join)
hash.update(col.name);
break;
case PROJECT:
for (const auto & pair_of_strs : action.projection)
{
hash.update(pair_of_strs.first);
hash.update(pair_of_strs.second);
}
break;
case ADD_ALIASES:
break;
}
return hash.get64();
}
bool ExpressionAction::operator==(const ExpressionAction & other) const
{
if (result_type != other.result_type)
{
if (result_type == nullptr || other.result_type == nullptr)
return false;
else if (!result_type->equals(*other.result_type))
return false;
}
if (function != other.function)
{
if (function == nullptr || other.function == nullptr)
return false;
else if (function->getName() != other.function->getName())
return false;
const auto & my_arg_types = function->getArgumentTypes();
const auto & other_arg_types = other.function->getArgumentTypes();
if (my_arg_types.size() != other_arg_types.size())
return false;
for (size_t i = 0; i < my_arg_types.size(); ++i)
if (!my_arg_types[i]->equals(*other_arg_types[i]))
return false;
}
if (added_column != other.added_column)
{
if (added_column == nullptr || other.added_column == nullptr)
return false;
else if (added_column->getName() != other.added_column->getName())
return false;
}
return source_name == other.source_name
&& result_name == other.result_name
&& row_projection_column == other.row_projection_column
&& is_row_projection_complementary == other.is_row_projection_complementary
&& argument_names == other.argument_names
&& array_joined_columns == other.array_joined_columns
&& array_join_is_left == other.array_join_is_left
&& join == other.join
&& join_key_names_left == other.join_key_names_left
&& columns_added_by_join == other.columns_added_by_join
&& projection == other.projection
&& is_function_compiled == other.is_function_compiled;
}
void ExpressionActionsChain::addStep()
{
if (steps.empty())
throw Exception("Cannot add action to empty ExpressionActionsChain", ErrorCodes::LOGICAL_ERROR);
ColumnsWithTypeAndName columns = steps.back().actions->getSampleBlock().getColumnsWithTypeAndName();
steps.push_back(Step(std::make_shared<ExpressionActions>(columns, settings)));
steps.push_back(Step(std::make_shared<ExpressionActions>(columns, context)));
}
void ExpressionActionsChain::finalize()

View File

@ -1,5 +1,8 @@
#pragma once
#include <Interpreters/Context.h>
#include <Common/config.h>
#include <Common/SipHash.h>
#include <Interpreters/Settings.h>
#include <Core/Names.h>
#include <Core/ColumnWithTypeAndName.h>
@ -83,6 +86,7 @@ public:
FunctionBuilderPtr function_builder;
FunctionBasePtr function;
Names argument_names;
bool is_function_compiled = false;
/// For ARRAY_JOIN
NameSet array_joined_columns;
@ -118,6 +122,13 @@ public:
std::string toString() const;
bool operator==(const ExpressionAction & other) const;
struct ActionHash
{
size_t operator()(const ExpressionAction & action) const;
};
private:
friend class ExpressionActions;
@ -135,16 +146,20 @@ class ExpressionActions
public:
using Actions = std::vector<ExpressionAction>;
ExpressionActions(const NamesAndTypesList & input_columns_, const Settings & settings_)
: input_columns(input_columns_), settings(settings_)
ExpressionActions(const NamesAndTypesList & input_columns_, const Context & context_)
: input_columns(input_columns_), settings(context_.getSettingsRef())
{
for (const auto & input_elem : input_columns)
sample_block.insert(ColumnWithTypeAndName(nullptr, input_elem.type, input_elem.name));
#if USE_EMBEDDED_COMPILER
compilation_cache = context_.getCompiledExpressionCache();
#endif
}
/// For constant columns the columns themselves can be contained in `input_columns_`.
ExpressionActions(const ColumnsWithTypeAndName & input_columns_, const Settings & settings_)
: settings(settings_)
ExpressionActions(const ColumnsWithTypeAndName & input_columns_, const Context & context_)
: settings(context_.getSettingsRef())
{
for (const auto & input_elem : input_columns_)
{
@ -213,11 +228,16 @@ public:
BlockInputStreamPtr createStreamWithNonJoinedDataIfFullOrRightJoin(const Block & source_header, size_t max_block_size) const;
const Settings & getSettings() const { return settings; }
private:
NamesAndTypesList input_columns;
Actions actions;
Block sample_block;
Settings settings;
#if USE_EMBEDDED_COMPILER
std::shared_ptr<CompiledExpressionCache> compilation_cache;
#endif
void checkLimits(Block & block) const;
@ -229,6 +249,18 @@ private:
using ExpressionActionsPtr = std::shared_ptr<ExpressionActions>;
struct ActionsHash
{
size_t operator()(const ExpressionActions::Actions & actions) const
{
SipHash hash;
for (const ExpressionAction & act : actions)
hash.update(ExpressionAction::ActionHash{}(act));
return hash.get64();
}
};
/** The sequence of transformations over the block.
* It is assumed that the result of each step is fed to the input of the next step.
@ -241,6 +273,8 @@ using ExpressionActionsPtr = std::shared_ptr<ExpressionActions>;
*/
struct ExpressionActionsChain
{
ExpressionActionsChain(const Context & context_)
: context(context_) {}
struct Step
{
ExpressionActionsPtr actions;
@ -259,7 +293,7 @@ struct ExpressionActionsChain
using Steps = std::vector<Step>;
Settings settings;
const Context & context;
Steps steps;
void addStep();

View File

@ -500,7 +500,7 @@ void ExpressionAnalyzer::analyzeAggregation()
if (select_query && (select_query->group_expression_list || select_query->having_expression))
has_aggregation = true;
ExpressionActionsPtr temp_actions = std::make_shared<ExpressionActions>(source_columns, settings);
ExpressionActionsPtr temp_actions = std::make_shared<ExpressionActions>(source_columns, context);
if (select_query && select_query->array_join_expression_list())
{
@ -1548,7 +1548,7 @@ void ExpressionAnalyzer::makeSetsForIndexImpl(const ASTPtr & node, const Block &
temp_columns.insert(temp_columns.end(), array_join_columns.begin(), array_join_columns.end());
for (const auto & joined_column : analyzed_join.columns_added_by_join)
temp_columns.push_back(joined_column.name_and_type);
ExpressionActionsPtr temp_actions = std::make_shared<ExpressionActions>(temp_columns, settings);
ExpressionActionsPtr temp_actions = std::make_shared<ExpressionActions>(temp_columns, context);
getRootActions(func->arguments->children.at(0), true, false, temp_actions);
Block sample_block_with_calculated_columns = temp_actions->getSampleBlock();
@ -1751,8 +1751,8 @@ static String getUniqueName(const Block & block, const String & prefix)
* For example, in the expression "select arrayMap(x -> x + column1 * column2, array1)"
* calculation of the product must be done outside the lambda expression (it does not depend on x), and the calculation of the sum is inside (depends on x).
*/
ScopeStack::ScopeStack(const ExpressionActionsPtr & actions, const Settings & settings_)
: settings(settings_)
ScopeStack::ScopeStack(const ExpressionActionsPtr & actions, const Context & context_)
: context(context_)
{
stack.emplace_back();
stack.back().actions = actions;
@ -1785,7 +1785,7 @@ void ScopeStack::pushLevel(const NamesAndTypesList & input_columns)
all_columns.push_back(col);
}
stack.back().actions = std::make_shared<ExpressionActions>(all_columns, settings);
stack.back().actions = std::make_shared<ExpressionActions>(all_columns, context);
}
size_t ScopeStack::getColumnLevel(const std::string & name)
@ -1831,7 +1831,7 @@ const Block & ScopeStack::getSampleBlock() const
void ExpressionAnalyzer::getRootActions(const ASTPtr & ast, bool no_subqueries, bool only_consts, ExpressionActionsPtr & actions)
{
ScopeStack scopes(actions, settings);
ScopeStack scopes(actions, context);
ProjectionManipulatorPtr projection_manipulator;
if (!isThereArrayJoin(ast) && settings.enable_conditional_computation && !only_consts)
@ -2002,7 +2002,7 @@ bool ExpressionAnalyzer::isThereArrayJoin(const ASTPtr & ast)
void ExpressionAnalyzer::getActionsFromJoinKeys(const ASTTableJoin & table_join, bool no_subqueries, bool only_consts,
ExpressionActionsPtr & actions)
{
ScopeStack scopes(actions, settings);
ScopeStack scopes(actions, context);
ProjectionManipulatorPtr projection_manipulator;
if (!isThereArrayJoin(query) && settings.enable_conditional_computation && !only_consts)
@ -2414,8 +2414,7 @@ void ExpressionAnalyzer::initChain(ExpressionActionsChain & chain, const NamesAn
{
if (chain.steps.empty())
{
chain.settings = settings;
chain.steps.emplace_back(std::make_shared<ExpressionActions>(columns, settings));
chain.steps.emplace_back(std::make_shared<ExpressionActions>(columns, context));
}
}
@ -2685,7 +2684,7 @@ bool ExpressionAnalyzer::appendPrewhere(ExpressionActionsChain & chain, bool onl
{
/// Remove unused source_columns from prewhere actions.
auto tmp_actions = std::make_shared<ExpressionActions>(source_columns, settings);
auto tmp_actions = std::make_shared<ExpressionActions>(source_columns, context);
getRootActions(select_query->prewhere_expression, only_types, false, tmp_actions);
tmp_actions->finalize({prewhere_column_name});
auto required_columns = tmp_actions->getRequiredColumns();
@ -2724,7 +2723,7 @@ bool ExpressionAnalyzer::appendPrewhere(ExpressionActionsChain & chain, bool onl
}
}
chain.steps.emplace_back(std::make_shared<ExpressionActions>(std::move(columns), settings));
chain.steps.emplace_back(std::make_shared<ExpressionActions>(std::move(columns), context));
chain.steps.back().additional_input = std::move(unused_source_columns);
}
@ -2905,7 +2904,7 @@ void ExpressionAnalyzer::getActionsBeforeAggregation(const ASTPtr & ast, Express
ExpressionActionsPtr ExpressionAnalyzer::getActions(bool add_aliases, bool project_result)
{
ExpressionActionsPtr actions = std::make_shared<ExpressionActions>(source_columns, settings);
ExpressionActionsPtr actions = std::make_shared<ExpressionActions>(source_columns, context);
NamesWithAliases result_columns;
Names result_names;
@ -2952,7 +2951,7 @@ ExpressionActionsPtr ExpressionAnalyzer::getActions(bool add_aliases, bool proje
ExpressionActionsPtr ExpressionAnalyzer::getConstActions()
{
ExpressionActionsPtr actions = std::make_shared<ExpressionActions>(NamesAndTypesList(), settings);
ExpressionActionsPtr actions = std::make_shared<ExpressionActions>(NamesAndTypesList(), context);
getRootActions(query, true, true, actions);

View File

@ -77,9 +77,10 @@ struct ScopeStack
using Levels = std::vector<Level>;
Levels stack;
const Settings & settings;
ScopeStack(const ExpressionActionsPtr & actions, const Settings & settings_);
const Context & context;
ScopeStack(const ExpressionActionsPtr & actions, const Context & context_);
void pushLevel(const NamesAndTypesList & input_columns);
@ -186,7 +187,7 @@ private:
ASTPtr query;
ASTSelectQuery * select_query;
const Context & context;
Settings settings;
const Settings settings;
size_t subquery_depth;
/** Original columns.

View File

@ -7,12 +7,14 @@
#include <Columns/ColumnConst.h>
#include <Columns/ColumnNullable.h>
#include <Columns/ColumnVector.h>
#include <Common/LRUCache.h>
#include <Common/MemoryTracker.h>
#include <Common/typeid_cast.h>
#include <Common/ProfileEvents.h>
#include <Common/Stopwatch.h>
#include <DataTypes/DataTypeNullable.h>
#include <DataTypes/DataTypesNumber.h>
#include <DataTypes/Native.h>
#include <Functions/IFunction.h>
#pragma GCC diagnostic push
#pragma GCC diagnostic ignored "-Wunused-parameter"
@ -49,6 +51,7 @@
namespace ProfileEvents
{
extern const Event CompileFunction;
extern const Event CompileExpressionsMicroseconds;
}
namespace DB
@ -160,8 +163,37 @@ auto wrapJITSymbolResolver(llvm::JITSymbolResolver & jsr)
}
#endif
#if LLVM_VERSION_MAJOR >= 6
struct CountingMMapper final : public llvm::SectionMemoryManager::MemoryMapper
{
MemoryTracker memory_tracker{VariableContext::Global};
llvm::sys::MemoryBlock allocateMappedMemory(llvm::SectionMemoryManager::AllocationPurpose /*purpose*/,
size_t num_bytes,
const llvm::sys::MemoryBlock * const near_block,
unsigned flags,
std::error_code & error_code) override
{
memory_tracker.alloc(num_bytes);
return llvm::sys::Memory::allocateMappedMemory(num_bytes, near_block, flags, error_code);
}
std::error_code protectMappedMemory(const llvm::sys::MemoryBlock & block, unsigned flags) override
{
return llvm::sys::Memory::protectMappedMemory(block, flags);
}
std::error_code releaseMappedMemory(llvm::sys::MemoryBlock & block) override
{
memory_tracker.free(block.size());
return llvm::sys::Memory::releaseMappedMemory(block);
}
};
#endif
struct LLVMContext
{
static inline std::atomic<size_t> id_counter{0};
llvm::LLVMContext context;
#if LLVM_VERSION_MAJOR >= 7
llvm::orc::ExecutionSession execution_session;
@ -170,12 +202,16 @@ struct LLVMContext
std::shared_ptr<llvm::Module> module;
#endif
std::unique_ptr<llvm::TargetMachine> machine;
#if LLVM_VERSION_MAJOR >= 6
std::unique_ptr<CountingMMapper> memory_mapper;
#endif
std::shared_ptr<llvm::SectionMemoryManager> memory_manager;
llvm::orc::RTDyldObjectLinkingLayer object_layer;
llvm::orc::IRCompileLayer<decltype(object_layer), llvm::orc::SimpleCompiler> compile_layer;
llvm::DataLayout layout;
llvm::IRBuilder<> builder;
std::unordered_map<std::string, void *> symbols;
size_t id;
LLVMContext()
#if LLVM_VERSION_MAJOR >= 7
@ -184,7 +220,13 @@ struct LLVMContext
: module(std::make_shared<llvm::Module>("jit", context))
#endif
, machine(getNativeMachine())
#if LLVM_VERSION_MAJOR >= 6
, memory_mapper(std::make_unique<CountingMMapper>())
, memory_manager(std::make_shared<llvm::SectionMemoryManager>(memory_mapper.get()))
#else
, memory_manager(std::make_shared<llvm::SectionMemoryManager>())
#endif
#if LLVM_VERSION_MAJOR >= 7
, object_layer(execution_session, [this](llvm::orc::VModuleKey)
{
@ -196,6 +238,7 @@ struct LLVMContext
, compile_layer(object_layer, llvm::orc::SimpleCompiler(*machine))
, layout(machine->createDataLayout())
, builder(context)
, id(id_counter++)
{
module->setDataLayout(layout);
module->setTargetTriple(machine->getTargetTriple().getTriple());
@ -290,7 +333,7 @@ public:
reinterpret_cast<void (*) (size_t, ColumnData *)>(function)(block_size, columns.data());
}
block.getByPosition(result).column = std::move(col_res);
};
}
};
static void compileFunction(std::shared_ptr<LLVMContext> & context, const IFunctionBase & f)
@ -424,126 +467,104 @@ static CompilableExpression subexpression(const IFunctionBase & f, std::vector<C
};
}
class LLVMFunction : public IFunctionBase
{
std::string name;
Names arg_names;
DataTypes arg_types;
std::shared_ptr<LLVMContext> context;
std::vector<FunctionBasePtr> originals;
std::unordered_map<StringRef, CompilableExpression> subexpressions;
public:
LLVMFunction(const ExpressionActions::Actions & actions, std::shared_ptr<LLVMContext> context, const Block & sample_block)
LLVMFunction::LLVMFunction(const ExpressionActions::Actions & actions, std::shared_ptr<LLVMContext> context, const Block & sample_block)
: name(actions.back().result_name), context(context)
{
for (const auto & c : sample_block)
/// TODO: implement `getNativeValue` for all types & replace the check with `c.column && toNativeType(...)`
if (c.column && getNativeValue(toNativeType(context->builder, c.type), *c.column, 0))
subexpressions[c.name] = subexpression(c.column, c.type);
for (const auto & action : actions)
{
for (const auto & c : sample_block)
/// TODO: implement `getNativeValue` for all types & replace the check with `c.column && toNativeType(...)`
if (c.column && getNativeValue(toNativeType(context->builder, c.type), *c.column, 0))
subexpressions[c.name] = subexpression(c.column, c.type);
for (const auto & action : actions)
const auto & names = action.argument_names;
const auto & types = action.function->getArgumentTypes();
std::vector<CompilableExpression> args;
for (size_t i = 0; i < names.size(); ++i)
{
const auto & names = action.argument_names;
const auto & types = action.function->getArgumentTypes();
std::vector<CompilableExpression> args;
for (size_t i = 0; i < names.size(); ++i)
auto inserted = subexpressions.emplace(names[i], subexpression(arg_names.size()));
if (inserted.second)
{
auto inserted = subexpressions.emplace(names[i], subexpression(arg_names.size()));
if (inserted.second)
{
arg_names.push_back(names[i]);
arg_types.push_back(types[i]);
}
args.push_back(inserted.first->second);
arg_names.push_back(names[i]);
arg_types.push_back(types[i]);
}
subexpressions[action.result_name] = subexpression(*action.function, std::move(args));
originals.push_back(action.function);
args.push_back(inserted.first->second);
}
compileFunction(context, *this);
subexpressions[action.result_name] = subexpression(*action.function, std::move(args));
originals.push_back(action.function);
}
compileFunction(context, *this);
}
bool isCompilable() const override { return true; }
PreparedFunctionPtr LLVMFunction::prepare(const Block &) const { return std::make_shared<LLVMPreparedFunction>(name, context); }
llvm::Value * compile(llvm::IRBuilderBase & builder, ValuePlaceholders values) const override { return subexpressions.at(name)(builder, values); }
bool LLVMFunction::isDeterministic() const
{
for (const auto & f : originals)
if (!f->isDeterministic())
return false;
return true;
}
String getName() const override { return name; }
bool LLVMFunction::isDeterministicInScopeOfQuery() const
{
for (const auto & f : originals)
if (!f->isDeterministicInScopeOfQuery())
return false;
return true;
}
const Names & getArgumentNames() const { return arg_names; }
bool LLVMFunction::isSuitableForConstantFolding() const
{
for (const auto & f : originals)
if (!f->isSuitableForConstantFolding())
return false;
return true;
}
const DataTypes & getArgumentTypes() const override { return arg_types; }
bool LLVMFunction::isInjective(const Block & sample_block)
{
for (const auto & f : originals)
if (!f->isInjective(sample_block))
return false;
return true;
}
const DataTypePtr & getReturnType() const override { return originals.back()->getReturnType(); }
bool LLVMFunction::hasInformationAboutMonotonicity() const
{
for (const auto & f : originals)
if (!f->hasInformationAboutMonotonicity())
return false;
return true;
}
PreparedFunctionPtr prepare(const Block &) const override { return std::make_shared<LLVMPreparedFunction>(name, context); }
bool isDeterministic() const override
LLVMFunction::Monotonicity LLVMFunction::getMonotonicityForRange(const IDataType & type, const Field & left, const Field & right) const
{
const IDataType * type_ = &type;
Field left_ = left;
Field right_ = right;
Monotonicity result(true, true, true);
/// monotonicity is only defined for unary functions, so the chain must describe a sequence of nested calls
for (size_t i = 0; i < originals.size(); ++i)
{
for (const auto & f : originals)
if (!f->isDeterministic())
return false;
return true;
}
bool isDeterministicInScopeOfQuery() const override
{
for (const auto & f : originals)
if (!f->isDeterministicInScopeOfQuery())
return false;
return true;
}
bool isSuitableForConstantFolding() const override
{
for (const auto & f : originals)
if (!f->isSuitableForConstantFolding())
return false;
return true;
}
bool isInjective(const Block & sample_block) override
{
for (const auto & f : originals)
if (!f->isInjective(sample_block))
return false;
return true;
}
bool hasInformationAboutMonotonicity() const override
{
for (const auto & f : originals)
if (!f->hasInformationAboutMonotonicity())
return false;
return true;
}
Monotonicity getMonotonicityForRange(const IDataType & type, const Field & left, const Field & right) const override
{
const IDataType * type_ = &type;
Field left_ = left;
Field right_ = right;
Monotonicity result(true, true, true);
/// monotonicity is only defined for unary functions, so the chain must describe a sequence of nested calls
for (size_t i = 0; i < originals.size(); ++i)
Monotonicity m = originals[i]->getMonotonicityForRange(*type_, left_, right_);
if (!m.is_monotonic)
return m;
result.is_positive ^= !m.is_positive;
result.is_always_monotonic &= m.is_always_monotonic;
if (i + 1 < originals.size())
{
Monotonicity m = originals[i]->getMonotonicityForRange(*type_, left_, right_);
if (!m.is_monotonic)
return m;
result.is_positive ^= !m.is_positive;
result.is_always_monotonic &= m.is_always_monotonic;
if (i + 1 < originals.size())
{
if (left_ != Field())
applyFunction(*originals[i], left_);
if (right_ != Field())
applyFunction(*originals[i], right_);
if (!m.is_positive)
std::swap(left_, right_);
type_ = originals[i]->getReturnType().get();
}
if (left_ != Field())
applyFunction(*originals[i], left_);
if (right_ != Field())
applyFunction(*originals[i], right_);
if (!m.is_positive)
std::swap(left_, right_);
type_ = originals[i]->getReturnType().get();
}
return result;
}
};
return result;
}
static bool isCompilable(llvm::IRBuilderBase & builder, const IFunctionBase & function)
{
@ -555,7 +576,29 @@ static bool isCompilable(llvm::IRBuilderBase & builder, const IFunctionBase & fu
return function.isCompilable();
}
void compileFunctions(ExpressionActions::Actions & actions, const Names & output_columns, const Block & sample_block)
size_t CompiledExpressionCache::weight() const
{
#if LLVM_VERSION_MAJOR >= 6
std::lock_guard<std::mutex> lock(mutex);
size_t result{0};
std::unordered_set<size_t> seen;
for (const auto & cell : cells)
{
auto function_context = cell.second.value->getContext();
if (!seen.count(function_context->id))
{
result += function_context->memory_mapper->memory_tracker.get();
seen.insert(function_context->id);
}
}
return result;
#else
return Base::weight();
#endif
}
void compileFunctions(ExpressionActions::Actions & actions, const Names & output_columns, const Block & sample_block, std::shared_ptr<CompiledExpressionCache> compilation_cache)
{
struct LLVMTargetInitializer
{
@ -638,9 +681,28 @@ void compileFunctions(ExpressionActions::Actions & actions, const Names & output
/// the result of compiling one function in isolation is pretty much the same as its `execute` method.
if (fused[i].size() == 1)
continue;
auto fn = std::make_shared<LLVMFunction>(std::move(fused[i]), context, sample_block);
std::shared_ptr<LLVMFunction> fn;
if (compilation_cache)
{
bool success;
auto set_func = [&fused, i, context, &sample_block] () { return std::make_shared<LLVMFunction>(fused[i], context, sample_block); };
Stopwatch watch;
std::tie(fn, success) = compilation_cache->getOrSet(fused[i], set_func);
if (success)
ProfileEvents::increment(ProfileEvents::CompileExpressionsMicroseconds, watch.elapsedMicroseconds());
}
else
{
Stopwatch watch;
fn = std::make_shared<LLVMFunction>(fused[i], context, sample_block);
ProfileEvents::increment(ProfileEvents::CompileExpressionsMicroseconds, watch.elapsedMicroseconds());
}
actions[i].function = fn;
actions[i].argument_names = fn->getArgumentNames();
actions[i].is_function_compiled = true;
continue;
}

View File

@ -4,14 +4,76 @@
#if USE_EMBEDDED_COMPILER
#include <Functions/IFunction.h>
#include <Interpreters/Context.h>
#include <Interpreters/ExpressionActions.h>
#include <Common/LRUCache.h>
#include <set>
namespace DB
{
struct LLVMContext;
using CompilableExpression = std::function<llvm::Value * (llvm::IRBuilderBase &, const ValuePlaceholders &)>;
class LLVMFunction : public IFunctionBase
{
std::string name;
Names arg_names;
DataTypes arg_types;
std::shared_ptr<LLVMContext> context;
std::vector<FunctionBasePtr> originals;
std::unordered_map<StringRef, CompilableExpression> subexpressions;
public:
LLVMFunction(const ExpressionActions::Actions & actions, std::shared_ptr<LLVMContext> context, const Block & sample_block);
bool isCompilable() const override { return true; }
llvm::Value * compile(llvm::IRBuilderBase & builder, ValuePlaceholders values) const override { return subexpressions.at(name)(builder, values); }
String getName() const override { return name; }
const Names & getArgumentNames() const { return arg_names; }
const DataTypes & getArgumentTypes() const override { return arg_types; }
const DataTypePtr & getReturnType() const override { return originals.back()->getReturnType(); }
PreparedFunctionPtr prepare(const Block &) const override;
bool isDeterministic() const override;
bool isDeterministicInScopeOfQuery() const override;
bool isSuitableForConstantFolding() const override;
bool isInjective(const Block & sample_block) override;
bool hasInformationAboutMonotonicity() const override;
Monotonicity getMonotonicityForRange(const IDataType & type, const Field & left, const Field & right) const override;
std::shared_ptr<LLVMContext> getContext() const { return context; }
};
/** This child of LRUCache breaks one of it's invariants: total weight may be changed after insertion.
* We have to do so, because we don't known real memory consumption of generated LLVM code for every function.
*/
class CompiledExpressionCache : public LRUCache<std::vector<ExpressionAction>, LLVMFunction, ActionsHash>
{
private:
using Base = LRUCache<std::vector<ExpressionAction>, LLVMFunction, ActionsHash>;
public:
using Base::Base;
size_t weight() const;
};
/// For each APPLY_FUNCTION action, try to compile the function to native code; if the only uses of a compilable
/// function's result are as arguments to other compilable functions, inline it and leave the now-redundant action as-is.
void compileFunctions(ExpressionActions::Actions & actions, const Names & output_columns, const Block & sample_block);
void compileFunctions(ExpressionActions::Actions & actions, const Names & output_columns, const Block & sample_block, std::shared_ptr<CompiledExpressionCache> compilation_cache);
}

View File

@ -370,7 +370,7 @@ void InterpreterCreateQuery::checkSupportedTypes(const ColumnsDescription & colu
+ "Set setting allow_experimental_low_cardinality_type = 1 in order to allow it.";
throw Exception(message, ErrorCodes::ILLEGAL_COLUMN);
}
if (!allow_decimal && column.type && isDecimal(*column.type))
if (!allow_decimal && column.type && isDecimal(column.type.get()))
{
String message = "Cannot create table with column '" + column.name + "' which type is '" + column.type->getName()
+ "'. Set setting allow_experimental_decimal_type = 1 in order to allow it.";

View File

@ -46,6 +46,7 @@
#include <Columns/Collator.h>
#include <Common/typeid_cast.h>
#include <Parsers/queryToString.h>
#include <ext/map.h>
namespace DB
@ -314,7 +315,7 @@ InterpreterSelectQuery::AnalysisResult InterpreterSelectQuery::analyzeExpression
};
{
ExpressionActionsChain chain;
ExpressionActionsChain chain(context);
if (query_analyzer->appendPrewhere(chain, !res.first_stage))
{
@ -677,6 +678,9 @@ void InterpreterSelectQuery::executeFetchColumns(
/// Separate expression for columns used in prewhere.
auto required_prewhere_columns_expr_list = std::make_shared<ASTExpressionList>();
/// Columns which we will get after prewhere execution.
auto source_columns = storage->getColumns().getAllPhysical();
auto physical_columns = ext::map<NameSet>(source_columns, [] (const auto & it) { return it.name; });
for (const auto & column : required_columns)
{
@ -723,19 +727,22 @@ void InterpreterSelectQuery::executeFetchColumns(
prewhere_info->remove_prewhere_column = false;
/// Remove columns which will be added by prewhere.
size_t next_req_column_pos = 0;
for (size_t i = 0; i < required_columns.size(); ++i)
{
if (!storage->getColumns().hasPhysical(required_columns[i]))
if (physical_columns.count(required_columns[i]))
{
std::swap(required_columns[i], required_columns.back());
required_columns.pop_back();
if (next_req_column_pos < i)
std::swap(required_columns[i], required_columns[next_req_column_pos]);
++next_req_column_pos;
}
}
required_columns.resize(next_req_column_pos);
if (prewhere_info)
{
/// Don't remove columns which are needed to be aliased.
auto new_actions = std::make_shared<ExpressionActions>(prewhere_info->prewhere_actions->getRequiredColumnsWithTypes(), settings);
auto new_actions = std::make_shared<ExpressionActions>(prewhere_info->prewhere_actions->getRequiredColumnsWithTypes(), context);
for (const auto & action : prewhere_info->prewhere_actions->getActions())
{
if (action.type != ExpressionAction::REMOVE_COLUMN

View File

@ -1,6 +1,7 @@
#include <Interpreters/InterpreterSystemQuery.h>
#include <Common/DNSResolver.h>
#include <Common/ActionLock.h>
#include <Common/config.h>
#include <Common/typeid_cast.h>
#include <Common/getNumberOfPhysicalCPUCores.h>
#include <common/ThreadPool.h>
@ -148,6 +149,11 @@ BlockIO InterpreterSystemQuery::execute()
case Type::DROP_UNCOMPRESSED_CACHE:
system_context.dropUncompressedCache();
break;
#if USE_EMBEDDED_COMPILER
case Type::DROP_COMPILED_EXPRESSION_CACHE:
system_context.dropCompiledExpressionCache();
break;
#endif
case Type::RELOAD_DICTIONARY:
system_context.getExternalDictionaries().reloadDictionary(query.target_dictionary);
break;

View File

@ -54,7 +54,7 @@ int main(int argc, char ** argv)
};
ExpressionAnalyzer analyzer(ast, context, {}, columns);
ExpressionActionsChain chain;
ExpressionActionsChain chain(context);
analyzer.appendSelect(chain, false);
analyzer.appendProjectResult(chain);
chain.finalize();

View File

@ -27,6 +27,10 @@ const char * ASTSystemQuery::typeToString(Type type)
return "DROP MARK CACHE";
case Type::DROP_UNCOMPRESSED_CACHE:
return "DROP UNCOMPRESSED CACHE";
#if USE_EMBEDDED_COMPILER
case Type::DROP_COMPILED_EXPRESSION_CACHE:
return "DROP COMPILED EXPRESSION CACHE";
#endif
case Type::STOP_LISTEN_QUERIES:
return "STOP LISTEN QUERIES";
case Type::START_LISTEN_QUERIES:

View File

@ -1,5 +1,6 @@
#pragma once
#include <Common/config.h>
#include <Parsers/IAST.h>
@ -18,6 +19,9 @@ public:
DROP_DNS_CACHE,
DROP_MARK_CACHE,
DROP_UNCOMPRESSED_CACHE,
#if USE_EMBEDDED_COMPILER
DROP_COMPILED_EXPRESSION_CACHE,
#endif
STOP_LISTEN_QUERIES,
START_LISTEN_QUERIES,
RESTART_REPLICAS,

View File

@ -272,7 +272,7 @@ void MergeTreeData::initPartitionKey()
/// Add all columns used in the partition key to the min-max index.
const NamesAndTypesList & minmax_idx_columns_with_types = partition_expr->getRequiredColumnsWithTypes();
minmax_idx_expr = std::make_shared<ExpressionActions>(minmax_idx_columns_with_types, context.getSettingsRef());
minmax_idx_expr = std::make_shared<ExpressionActions>(minmax_idx_columns_with_types, context);
for (const NameAndTypePair & column : minmax_idx_columns_with_types)
{
minmax_idx_columns.emplace_back(column.name);
@ -996,7 +996,7 @@ void MergeTreeData::createConvertExpression(const DataPartPtr & part, const Name
/// Need to modify column type.
if (!out_expression)
out_expression = std::make_shared<ExpressionActions>(NamesAndTypesList(), context.getSettingsRef());
out_expression = std::make_shared<ExpressionActions>(NamesAndTypesList(), context);
out_expression->addInput(ColumnWithTypeAndName(nullptr, column.type, column.name));

View File

@ -63,6 +63,16 @@ StorageBuffer::StorageBuffer(const std::string & name_, const ColumnsDescription
{
}
StorageBuffer::~StorageBuffer()
{
// Should not happen if shutdown was called
if (flush_thread.joinable())
{
shutdown_event.set();
flush_thread.join();
}
}
/// Reads from one buffer (from one block) under its mutex.
class BufferBlockInputStream : public IProfilingBlockInputStream

View File

@ -131,6 +131,7 @@ protected:
Context & context_,
size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_,
const String & destination_database_, const String & destination_table_, bool allow_materialized_);
~StorageBuffer();
};
}

View File

@ -47,7 +47,7 @@ if [ "$DATA_DIR_PATTERN" != "$DATA_DIR" ]; then
fi
CLICKHOUSE_EXTRACT_CONFIG=${CLICKHOUSE_EXTRACT_CONFIG:="${BIN_DIR}${CLICKHOUSE_BINARY}-extract-from-config --config=$CLICKHOUSE_CONFIG"}
CLICKHOUSE_LOG=${CLICKHOUSE_LOG:=$DATA_DIR/log/clickhouse-server.log}
CLICKHOUSE_LOG=${CLICKHOUSE_LOG:=${LOG_DIR}clickhouse-server.log}
export CLICKHOUSE_PORT_TCP=${CLICKHOUSE_PORT_TCP:=`$CLICKHOUSE_EXTRACT_CONFIG --key=tcp_port || echo 9000`}
export CLICKHOUSE_PORT_HTTP=${CLICKHOUSE_PORT_HTTP:=`$CLICKHOUSE_EXTRACT_CONFIG --key=http_port || echo 8123`}
export CLICKHOUSE_PORT_INTERSERVER=${CLICKHOUSE_PORT_INTERSERVER:=`$CLICKHOUSE_EXTRACT_CONFIG --key=interserver_http_port || echo 9009`}
@ -63,7 +63,7 @@ CERT=`${BIN_DIR}clickhouse-extract-from-config --config=$CLICKHOUSE_CONFIG --key
[ -n "$PRIVATEKEY" ] && [ -n "$CERT" ] && openssl req -subj "/CN=localhost" -new -newkey rsa:2048 -days 365 -nodes -x509 -keyout $PRIVATEKEY -out $CERT
if [ "$TEST_GDB" ] || [ "$GDB" ]; then
echo -e "run \nset pagination off \nset logging file $DATA_DIR/gdb.log \nset logging on \nthread apply all backtrace \ndetach \nquit " > $DATA_DIR/gdb.cmd
echo -e "run \nset pagination off \nset logging file $LOG_DIR/server.gdb.log \nset logging on \nbacktrace \nthread apply all backtrace \nbacktrace \ndetach \nquit " > $DATA_DIR/gdb.cmd
GDB=${GDB:="gdb -x $DATA_DIR/gdb.cmd --args "}
fi
@ -92,7 +92,7 @@ sleep ${TEST_SERVER_STARTUP_WAIT:=5}
if [ "$GDB" ]; then
# Long symbols read
sleep 40
sleep ${TEST_GDB_SLEEP:=60}
fi
tail -n50 $LOG_DIR/*.log || true
@ -103,7 +103,7 @@ function finish {
wait
tail -n 50 $LOG_DIR/*.log || true
if [ "$GDB" ]; then
cat $DATA_DIR/gdb.log || true
cat $LOG_DIR/server.gdb.log || true
fi
rm -rf $DATA_DIR
}
@ -115,7 +115,20 @@ if [ -n "$*" ]; then
else
TEST_RUN=${TEST_RUN=1}
TEST_PERF=${TEST_PERF=1}
${BIN_DIR}clickhouse-client --config ${CLICKHOUSE_CONFIG_CLIENT} --port $CLICKHOUSE_PORT_TCP -m -n -q 'SELECT * from system.build_options; SELECT * FROM system.clusters;'
[ "$TEST_RUN" ] && env PATH=$PATH:$BIN_DIR ${TEST_DIR}clickhouse-test --binary ${BIN_DIR}clickhouse --configclient $CLICKHOUSE_CONFIG_CLIENT --configserver $CLICKHOUSE_CONFIG --tmp $DATA_DIR/tmp --queries $QUERIES_DIR $TEST_OPT0 $TEST_OPT
( [ "$TEST_PERF" ] && ${BIN_DIR}clickhouse-performance-test --port $CLICKHOUSE_PORT_TCP --r $CUR_DIR/performance --skip-tags=long $* ) || true
CLICKHOUSE_CLIENT_QUERY="${BIN_DIR}clickhouse-client --config ${CLICKHOUSE_CONFIG_CLIENT} --port $CLICKHOUSE_PORT_TCP -m -n -q"
$CLICKHOUSE_CLIENT_QUERY 'SELECT * from system.build_options; SELECT * FROM system.clusters;'
CLICKHOUSE_TEST="env PATH=$PATH:$BIN_DIR ${TEST_DIR}clickhouse-test --binary ${BIN_DIR}clickhouse --configclient $CLICKHOUSE_CONFIG_CLIENT --configserver $CLICKHOUSE_CONFIG --tmp $DATA_DIR/tmp --queries $QUERIES_DIR $TEST_OPT0 $TEST_OPT"
CLICKHOUSE_PERFORMANCE_TEST="${BIN_DIR}clickhouse-performance-test --port $CLICKHOUSE_PORT_TCP --r $CUR_DIR/performance --skip-tags=long"
if [ "${TEST_RUN_STRESS}" ]; then
# Running test in parallel will fail some results (tests can create/fill/drop same tables)
TEST_NPROC=${TEST_NPROC:=$(( `nproc || sysctl -n hw.ncpu || echo 2` * 2))}
for i in `seq 1 ${TEST_NPROC}`; do
$CLICKHOUSE_TEST --order=random --testname &
done
$CLICKHOUSE_PERFORMANCE_TEST &
fi
( [ "$TEST_RUN" ] && $CLICKHOUSE_TEST ) || ${TEST_TRUE:=false}
( [ "$TEST_PERF" ] && $CLICKHOUSE_PERFORMANCE_TEST $* ) || true
$CLICKHOUSE_CLIENT_QUERY "SELECT * FROM system.events; SELECT * FROM system.metrics; SELECT * FROM asynchronous_metrics;"
$CLICKHOUSE_CLIENT_QUERY "SELECT 'Still alive'"
fi

View File

@ -188,23 +188,23 @@ class ClickHouseCluster:
self.docker_client = docker.from_env(version=self.docker_api_version)
if self.with_zookeeper and self.base_zookeeper_cmd:
subprocess.check_call(self.base_zookeeper_cmd + ['up', '-d', '--no-recreate'])
subprocess.check_call(self.base_zookeeper_cmd + ['up', '-d', '--force-recreate', '--remove-orphans'])
for command in self.pre_zookeeper_commands:
self.run_kazoo_commands_with_retries(command, repeats=5)
self.wait_zookeeper_to_start()
self.wait_zookeeper_to_start(120)
if self.with_mysql and self.base_mysql_cmd:
subprocess.check_call(self.base_mysql_cmd + ['up', '-d', '--no-recreate'])
subprocess.check_call(self.base_mysql_cmd + ['up', '-d', '--force-recreate', '--remove-orphans'])
self.wait_mysql_to_start(120)
if self.with_kafka and self.base_kafka_cmd:
subprocess.check_call(self.base_kafka_cmd + ['up', '-d', '--no-recreate'])
subprocess.check_call(self.base_kafka_cmd + ['up', '-d', '--force-recreate', '--remove-orphans'])
self.kafka_docker_id = self.get_instance_docker_id('kafka1')
# Uncomment for debugging
#print ' '.join(self.base_cmd + ['up', '--no-recreate'])
subprocess.check_call(self.base_cmd + ['up', '-d', '--no-recreate'])
subprocess.check_call(self.base_cmd + ['up', '-d', '--force-recreate', '--remove-orphans'])
start_deadline = time.time() + 20.0 # seconds
for instance in self.instances.itervalues():

View File

@ -50,7 +50,7 @@ def insert_reliable(instance, query_insert):
raise last_exception
TEST_REPLICATED_ALTERS=True
TEST_REPLICATED_ALTERS=False # TODO: Check code and turn on
cluster = ClickHouseCluster(__file__)
@ -196,7 +196,7 @@ def test_on_session_expired(started_cluster):
def test_replicated_alters(started_cluster):
instance = cluster.instances['ch2']
ddl_check_query(instance, "DROP TABLE IF EXISTS merge ON CLUSTER cluster")
ddl_check_query(instance, "DROP TABLE IF EXISTS merge_for_alter ON CLUSTER cluster")
ddl_check_query(instance, "DROP TABLE IF EXISTS all_merge_32 ON CLUSTER cluster")
ddl_check_query(instance, "DROP TABLE IF EXISTS all_merge_64 ON CLUSTER cluster")
@ -207,43 +207,43 @@ def test_replicated_alters(started_cluster):
firewall_drops_rules = cluster.pm_random_drops.pop_rules()
ddl_check_query(instance, """
CREATE TABLE IF NOT EXISTS merge ON CLUSTER cluster (p Date, i Int32)
CREATE TABLE IF NOT EXISTS merge_for_alter ON CLUSTER cluster (p Date, i Int32)
ENGINE = ReplicatedMergeTree('/clickhouse/tables/{layer}-{shard}/hits', '{replica}', p, p, 1)
""")
ddl_check_query(instance, """
CREATE TABLE IF NOT EXISTS all_merge_32 ON CLUSTER cluster (p Date, i Int32)
ENGINE = Distributed(cluster, default, merge, i)
ENGINE = Distributed(cluster, default, merge_for_alter, i)
""")
ddl_check_query(instance, """
CREATE TABLE IF NOT EXISTS all_merge_64 ON CLUSTER cluster (p Date, i Int64, s String)
ENGINE = Distributed(cluster, default, merge, i)
ENGINE = Distributed(cluster, default, merge_for_alter, i)
""")
for i in xrange(4):
k = (i / 2) * 2
insert_reliable(cluster.instances['ch{}'.format(i + 1)], "INSERT INTO merge (i) VALUES ({})({})".format(k, k+1))
insert_reliable(cluster.instances['ch{}'.format(i + 1)], "INSERT INTO merge_for_alter (i) VALUES ({})({})".format(k, k+1))
assert TSV(instance.query("SELECT i FROM all_merge_32 ORDER BY i")) == TSV(''.join(['{}\n'.format(x) for x in xrange(4)]))
ddl_check_query(instance, "ALTER TABLE merge ON CLUSTER cluster MODIFY COLUMN i Int64")
ddl_check_query(instance, "ALTER TABLE merge ON CLUSTER cluster ADD COLUMN s DEFAULT toString(i)")
ddl_check_query(instance, "ALTER TABLE merge_for_alter ON CLUSTER cluster MODIFY COLUMN i Int64")
ddl_check_query(instance, "ALTER TABLE merge_for_alter ON CLUSTER cluster ADD COLUMN s DEFAULT toString(i)")
assert TSV(instance.query("SELECT i, s FROM all_merge_64 ORDER BY i")) == TSV(''.join(['{}\t{}\n'.format(x,x) for x in xrange(4)]))
for i in xrange(4):
k = (i / 2) * 2 + 4
insert_reliable(cluster.instances['ch{}'.format(i + 1)], "INSERT INTO merge (p, i) VALUES (31, {})(31, {})".format(k, k+1))
insert_reliable(cluster.instances['ch{}'.format(i + 1)], "INSERT INTO merge_for_alter (p, i) VALUES (31, {})(31, {})".format(k, k+1))
assert TSV(instance.query("SELECT i, s FROM all_merge_64 ORDER BY i")) == TSV(''.join(['{}\t{}\n'.format(x,x) for x in xrange(8)]))
ddl_check_query(instance, "ALTER TABLE merge ON CLUSTER cluster DETACH PARTITION 197002")
ddl_check_query(instance, "ALTER TABLE merge_for_alter ON CLUSTER cluster DETACH PARTITION 197002")
assert TSV(instance.query("SELECT i, s FROM all_merge_64 ORDER BY i")) == TSV(''.join(['{}\t{}\n'.format(x,x) for x in xrange(4)]))
ddl_check_query(instance, "DROP TABLE merge ON CLUSTER cluster")
ddl_check_query(instance, "DROP TABLE merge_for_alter ON CLUSTER cluster")
# Enable random ZK packet drops
cluster.pm_random_drops.push_rules(firewall_drops_rules)

View File

@ -135,8 +135,8 @@ def test_insert_multithreaded(started_cluster):
assert runner.total_inserted > 0
all_replicated = False
for i in range(50): # wait for replication 5 seconds max
time.sleep(0.1)
for i in range(100): # wait for replication 50 seconds max
time.sleep(0.5)
def get_delay(node):
return int(node.query("SELECT absolute_delay FROM system.replicas WHERE table = 'repl_test'").rstrip())

View File

@ -155,8 +155,8 @@ def test_mutations(started_cluster):
assert runner.total_mutations > 0
all_done = False
for i in range(100): # wait for replication 10 seconds max
time.sleep(0.1)
for i in range(100): # wait for replication 50 seconds max
time.sleep(0.5)
def get_done_mutations(node):
return int(node.query("SELECT sum(is_done) FROM system.mutations WHERE table = 'test_mutations'").rstrip())

View File

@ -0,0 +1,47 @@
<test>
<name>small_requests</name>
<type>loop</type>
<stop_conditions>
<all_of>
<iterations>5</iterations>
<min_time_not_changing_for_ms>10000</min_time_not_changing_for_ms>
</all_of>
<any_of>
<iterations>5000</iterations>
<total_time_ms>60000</total_time_ms>
</any_of>
</stop_conditions>
<main_metric>
<bytes_per_second/>
</main_metric>
<metrics>
<rows_per_second/>
</metrics>
<query>
WITH
bitXor(number, 0x4CF2D2BAAE6DA887) AS x0,
bitXor(x0, bitShiftRight(x0, 33)) AS x1,
x1 * 0xff51afd7ed558ccd AS x2,
bitXor(x2, bitShiftRight(x2, 33)) AS x3,
x3 * 0xc4ceb9fe1a85ec53 AS x4,
bitXor(x4, bitShiftRight(x4, 33)) AS x5
SELECT x5, intHash64(number) FROM system.numbers LIMIT 10
</query>
<query>
WITH
bitXor(number, 0x4CF2D2BAAE6DA887) AS x0,
bitXor(x0, bitShiftRight(x0, 33)) AS x1,
x1 * 0xff51afd7ed558ccd AS x2,
bitXor(x2, bitShiftRight(x2, 33)) AS x3,
x3 * 0xc4ceb9fe1a85ec53 AS x4,
bitXor(x4, bitShiftRight(x4, 33)) AS x5
SELECT x5, intHash64(number) FROM system.numbers LIMIT 10
SETTINGS
compile = 1,
compile_expressions = 1
</query>
</test>

View File

@ -1,6 +1,6 @@
#!/usr/bin/env bash
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
. $CURDIR/../shell_config.sh
[ "$NO_SHELL_CONFIG" ] || . $CURDIR/../shell_config.sh
seq 1 1000 | sed -r 's/.+/CREATE TABLE IF NOT EXISTS test.buf (a UInt8) ENGINE = Buffer(test, b, 1, 1, 1, 1, 1, 1, 1); DROP TABLE test.buf;/' | $CLICKHOUSE_CLIENT -n

View File

@ -0,0 +1,14 @@
#!/usr/bin/env bash
CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
. $CURDIR/../shell_config.sh
export NO_SHELL_CONFIG=1
for i in {1..4}; do
$CURDIR/00097_long_storage_buffer_race_condition.sh > /dev/null 2>&1 &
done
wait
$CLICKHOUSE_CLIENT -q "SELECT 'Still alive'";

View File

@ -0,0 +1,56 @@
Array(Decimal(9, 3)) Array(Decimal(18, 3)) Array(Decimal(38, 3))
Array(Decimal(9, 2)) Array(Decimal(18, 2)) Array(Decimal(38, 2))
Decimal(9, 3) Decimal(18, 3) Decimal(38, 3)
Decimal(9, 2) Decimal(18, 2) Decimal(38, 2)
Tuple(Decimal(9, 1), Decimal(18, 1), Decimal(38, 1)) Decimal(9, 1) Decimal(18, 1) Decimal(38, 1)
0.100
0.200
0.300
0.400
0.500
0.600
0.700
0.800
0.900
(9.1,9.2,9.3) 9.1 9.2 9.3
[0.100,0.200,0.300] [0.100,0.200] [0.200,0.300] [0.100] [0.200]
[0.400,0.500,0.600] [0.400,0.500] [0.500,0.600] [0.400] [0.500]
[0.700,0.800,0.900] [0.700,0.800] [0.800,0.900] [0.700] [0.800]
[1.10,1.20] [1.10] [1.20] [1.10] [1.20]
[2.10,2.20] [2.10] [2.20] [2.10] [2.20]
[3.10,3.20] [3.10] [3.20] [3.10] [3.20]
[0.100,0.200,0.300,0.000] [0.000,0.100,0.200,0.300]
[0.400,0.500,0.600,0.000] [0.000,0.400,0.500,0.600]
[0.700,0.800,0.900,0.000] [0.000,0.700,0.800,0.900]
[0.100,0.200,0.300,0.000] Array(Decimal(9, 3))
[0.400,0.500,0.600,0.000] Array(Decimal(18, 3))
[0.700,0.800,0.900,0.000] Array(Decimal(38, 3))
[0.0000,0.1000,0.2000,0.3000] Array(Decimal(9, 4))
[0.0000,0.4000,0.5000,0.6000] Array(Decimal(18, 4))
[0.0000,0.7000,0.8000,0.9000] Array(Decimal(38, 4))
3 3 3
2 2 2
0 0 0
0 0 0
1 1 1
1 1 1
3 3 3
2 2 2
1 0
1 0
1 0
1 0
2 0
3 0
[0.100,0.200,0.300,0.400,0.500,0.600] Array(Decimal(18, 3))
[0.100,0.200,0.300,0.700,0.800,0.900] Array(Decimal(38, 3))
[0.400,0.500,0.600,0.700,0.800,0.900] Array(Decimal(38, 3))
[0.100,0.200,0.300,1.100,1.200] Array(Decimal(9, 3))
[0.400,0.500,0.600,2.100,2.200] Array(Decimal(18, 3))
[0.700,0.800,0.900,3.100,3.200] Array(Decimal(38, 3))
[0.100,0.200,0.300,2.100,2.200] Array(Decimal(18, 3))
[0.100,0.200,0.300,3.100,3.200] Array(Decimal(38, 3))
[0.400,0.500,0.600,1.100,1.200] Array(Decimal(18, 3))
[0.400,0.500,0.600,3.100,3.200] Array(Decimal(38, 3))
[0.700,0.800,0.900,1.100,1.200] Array(Decimal(38, 3))
[0.700,0.800,0.900,2.100,2.200] Array(Decimal(38, 3))

View File

@ -0,0 +1,108 @@
SET allow_experimental_decimal_type = 1;
SET send_logs_level = 'none';
CREATE DATABASE IF NOT EXISTS test;
DROP TABLE IF EXISTS test.decimal;
CREATE TABLE test.decimal
(
a Array(Decimal32(3)),
b Array(Decimal64(3)),
c Array(Decimal128(3)),
nest Nested
(
a Decimal(9,2),
b Decimal(18,2),
c Decimal(38,2)
),
tup Tuple(Decimal32(1), Decimal64(1), Decimal128(1))
) ENGINE = Memory;
INSERT INTO test.decimal (a, b, c, nest.a, nest.b, nest.c, tup)
VALUES ([0.1, 0.2, 0.3], [0.4, 0.5, 0.6], [0.7, 0.8, 0.9], [1.1, 1.2], [2.1, 2.2], [3.1, 3.2], (9.1, 9.2, 9.3));
SELECT toTypeName(a), toTypeName(b), toTypeName(c) FROM test.decimal;
SELECT toTypeName(nest.a), toTypeName(nest.b), toTypeName(nest.c) FROM test.decimal;
SELECT toTypeName(a[1]), toTypeName(b[2]), toTypeName(c[3]) FROM test.decimal;
SELECT toTypeName(nest.a[1]), toTypeName(nest.b[1]), toTypeName(nest.c[1]) FROM test.decimal;
SELECT toTypeName(tup), toTypeName(tup.1), toTypeName(tup.2), toTypeName(tup.3) FROM test.decimal;
SELECT arrayJoin(a) FROM test.decimal;
SELECT arrayJoin(b) FROM test.decimal;
SELECT arrayJoin(c) FROM test.decimal;
SELECT tup, tup.1, tup.2, tup.3 FROM test.decimal;
SELECT a, arrayPopBack(a), arrayPopFront(a), arrayResize(a, 1), arraySlice(a, 2, 1) FROM test.decimal;
SELECT b, arrayPopBack(b), arrayPopFront(b), arrayResize(b, 1), arraySlice(b, 2, 1) FROM test.decimal;
SELECT c, arrayPopBack(c), arrayPopFront(c), arrayResize(c, 1), arraySlice(c, 2, 1) FROM test.decimal;
SELECT nest.a, arrayPopBack(nest.a), arrayPopFront(nest.a), arrayResize(nest.a, 1), arraySlice(nest.a, 2, 1) FROM test.decimal;
SELECT nest.b, arrayPopBack(nest.b), arrayPopFront(nest.b), arrayResize(nest.b, 1), arraySlice(nest.b, 2, 1) FROM test.decimal;
SELECT nest.c, arrayPopBack(nest.c), arrayPopFront(nest.c), arrayResize(nest.c, 1), arraySlice(nest.c, 2, 1) FROM test.decimal;
SELECT arrayPushBack(a, toDecimal32(0, 3)), arrayPushFront(a, toDecimal32(0, 3)) FROM test.decimal;
SELECT arrayPushBack(b, toDecimal64(0, 3)), arrayPushFront(b, toDecimal64(0, 3)) FROM test.decimal;
SELECT arrayPushBack(c, toDecimal128(0, 3)), arrayPushFront(c, toDecimal128(0, 3)) FROM test.decimal;
SELECT arrayPushBack(a, toDecimal32(0, 2)) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayPushBack(b, toDecimal64(0, 2)) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayPushBack(c, toDecimal128(0, 2)) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayPushFront(a, toDecimal32(0, 4)) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayPushFront(b, toDecimal64(0, 4)) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayPushFront(c, toDecimal128(0, 4)) AS x, toTypeName(x) FROM test.decimal;
SELECT length(a), length(b), length(c) FROM test.decimal;
SELECT length(nest.a), length(nest.b), length(nest.c) FROM test.decimal;
SELECT empty(a), empty(b), empty(c) FROM test.decimal;
SELECT empty(nest.a), empty(nest.b), empty(nest.c) FROM test.decimal;
SELECT notEmpty(a), notEmpty(b), notEmpty(c) FROM test.decimal;
SELECT notEmpty(nest.a), notEmpty(nest.b), notEmpty(nest.c) FROM test.decimal;
SELECT arrayUniq(a), arrayUniq(b), arrayUniq(c) FROM test.decimal;
SELECT arrayUniq(nest.a), arrayUniq(nest.b), arrayUniq(nest.c) FROM test.decimal;
SELECT has(a, toDecimal32(0.1, 3)), has(a, toDecimal32(1.0, 3)) FROM test.decimal;
SELECT has(b, toDecimal64(0.4, 3)), has(b, toDecimal64(1.0, 3)) FROM test.decimal;
SELECT has(c, toDecimal128(0.7, 3)), has(c, toDecimal128(1.0, 3)) FROM test.decimal;
SELECT has(a, toDecimal32(0.1, 2)) FROM test.decimal; -- { serverError 43 }
SELECT has(a, toDecimal32(0.1, 4)) FROM test.decimal; -- { serverError 43 }
SELECT has(a, toDecimal64(0.1, 3)) FROM test.decimal; -- { serverError 43 }
SELECT has(a, toDecimal128(0.1, 3)) FROM test.decimal; -- { serverError 43 }
SELECT has(b, toDecimal32(0.4, 3)) FROM test.decimal; -- { serverError 43 }
SELECT has(b, toDecimal64(0.4, 2)) FROM test.decimal; -- { serverError 43 }
SELECT has(b, toDecimal64(0.4, 4)) FROM test.decimal; -- { serverError 43 }
SELECT has(b, toDecimal128(0.4, 3)) FROM test.decimal; -- { serverError 43 }
SELECT has(c, toDecimal32(0.7, 3)) FROM test.decimal; -- { serverError 43 }
SELECT has(c, toDecimal64(0.7, 3)) FROM test.decimal; -- { serverError 43 }
SELECT has(c, toDecimal128(0.7, 2)) FROM test.decimal; -- { serverError 43 }
SELECT has(c, toDecimal128(0.7, 4)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(a, toDecimal32(0.1, 3)), indexOf(a, toDecimal32(1.0, 3)) FROM test.decimal;
SELECT indexOf(b, toDecimal64(0.5, 3)), indexOf(b, toDecimal64(1.0, 3)) FROM test.decimal;
SELECT indexOf(c, toDecimal128(0.9, 3)), indexOf(c, toDecimal128(1.0, 3)) FROM test.decimal;
SELECT indexOf(a, toDecimal32(0.1, 2)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(a, toDecimal32(0.1, 4)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(a, toDecimal64(0.1, 3)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(a, toDecimal128(0.1, 3)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(b, toDecimal32(0.4, 3)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(b, toDecimal64(0.4, 2)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(b, toDecimal64(0.4, 4)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(b, toDecimal128(0.4, 3)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(c, toDecimal32(0.7, 3)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(c, toDecimal64(0.7, 3)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(c, toDecimal128(0.7, 2)) FROM test.decimal; -- { serverError 43 }
SELECT indexOf(c, toDecimal128(0.7, 4)) FROM test.decimal; -- { serverError 43 }
SELECT arrayConcat(a, b) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(a, c) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(b, c) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(a, nest.a) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(b, nest.b) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(c, nest.c) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(a, nest.b) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(a, nest.c) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(b, nest.a) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(b, nest.c) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(c, nest.a) AS x, toTypeName(x) FROM test.decimal;
SELECT arrayConcat(c, nest.b) AS x, toTypeName(x) FROM test.decimal;
DROP TABLE IF EXISTS test.decimal;

View File

@ -14,12 +14,12 @@ ORDER BY (d2, d3);
INSERT INTO test.decimal (d1, d2, d3) VALUES (4.2, 4.2, 4.2);
SELECT count() FROM test.decimal WHERE d1 = toDecimal32('4.2', 1);
SELECT count() FROM test.decimal WHERE d1 != toDecimal32('4.2', 2);
SELECT count() FROM test.decimal WHERE d1 < toDecimal32('4.2', 3);
SELECT count() FROM test.decimal WHERE d1 > toDecimal32('4.2', 4);
SELECT count() FROM test.decimal WHERE d1 <= toDecimal32('4.2', 5);
SELECT count() FROM test.decimal WHERE d1 >= toDecimal32('4.2', 6);
SELECT count() FROM test.decimal WHERE d1 = toDecimal32('4.2', 8);
SELECT count() FROM test.decimal WHERE d1 != toDecimal32('4.2', 8);
SELECT count() FROM test.decimal WHERE d1 < toDecimal32('4.2', 8);
SELECT count() FROM test.decimal WHERE d1 > toDecimal32('4.2', 8);
SELECT count() FROM test.decimal WHERE d1 <= toDecimal32('4.2', 8);
SELECT count() FROM test.decimal WHERE d1 >= toDecimal32('4.2', 8);
INSERT INTO test.decimal (d1, d2, d3)
SELECT toDecimal32(number % 10, 8), toDecimal64(number, 8), toDecimal128(number, 8) FROM system.numbers LIMIT 50;

View File

@ -0,0 +1,5 @@
drop table if exists test.prewhere_alias;
create table test.prewhere_alias (a Int32, b Int32, c alias a + b) engine = MergeTree order by b;
insert into test.prewhere_alias values(1, 1);
select a, c + toInt32(1), (c + toInt32(1)) * 2 from test.prewhere_alias prewhere (c + toInt32(1)) * 2 = 6;
drop table test.prewhere_alias;

4
debian/changelog vendored
View File

@ -1,5 +1,5 @@
clickhouse (18.12.5) unstable; urgency=low
clickhouse (18.12.6) unstable; urgency=low
* Modified source code
-- <root@yandex-team.ru> Thu, 06 Sep 2018 07:25:55 +0300
-- <root@yandex-team.ru> Thu, 06 Sep 2018 15:16:36 +0300

View File

@ -1,7 +1,7 @@
FROM ubuntu:18.04
ARG repository="deb http://repo.yandex.ru/clickhouse/deb/stable/ main/"
ARG version=18.12.5
ARG version=18.12.6
RUN apt-get update && \
apt-get install -y apt-transport-https dirmngr && \

View File

@ -1,7 +1,7 @@
FROM ubuntu:18.04
ARG repository="deb http://repo.yandex.ru/clickhouse/deb/stable/ main/"
ARG version=18.12.5
ARG version=18.12.6
RUN apt-get update && \
apt-get install -y apt-transport-https dirmngr && \

View File

@ -1,7 +1,7 @@
FROM ubuntu:18.04
ARG repository="deb http://repo.yandex.ru/clickhouse/deb/stable/ main/"
ARG version=18.12.5
ARG version=18.12.6
RUN apt-get update && \
apt-get install -y apt-transport-https dirmngr && \

1
docs/fa/changelog.md Symbolic link
View File

@ -0,0 +1 @@
../../CHANGELOG_RU.md

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Array(T)

View File

@ -1,7 +1,7 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# مقادیر Boolean
type مخصوص مقادیر boolean وجود ندارد. از Uint8 و محدود شده به 0 و 1 می توان استفاده کرد.
</div>
</div>

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Date

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# DateTime

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Enum
@ -12,7 +12,7 @@ Enum8 یا Enum16، به شما اجازه ی ذخیره سازی مجموعه
Enum8('hello' = 1, 'world' = 2)
```
<div dir="rtl">
<div dir="rtl" markdown="1">
- مقدار داخل این ستون می تواند یکی از دو مقدار 'hello' یا 'world' باشد.

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# FixedString(N)

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Float32, Float64
@ -26,7 +26,7 @@ SELECT 1 - 0.9
└─────────────────────┘
```
<div dir="rtl">
<div dir="rtl" markdown="1">
- نتایج محاسبات بسته به متد محاسباتی می باشد (نوع processor و معماری سیستم).
- محاسبات Float ممکن اسن نتایجی مثل infinity (`inf`) و "Not-a-number" (`Nan`) داشته باشد. این در هنگام پردازش نتایج محاسبات باید مورد توجه قرار گیرد.
@ -50,7 +50,7 @@ SELECT 0.5 / 0
└────────────────┘
```
<div dir="rtl">
<div dir="rtl" markdown="1">
- `-Inf` Negative infinity.
@ -66,7 +66,7 @@ SELECT -0.5 / 0
└─────────────────┘
```
<div dir="rtl">
<div dir="rtl" markdown="1">
- `NaN` Not a number.
@ -82,7 +82,7 @@ SELECT 0 / 0
└──────────────┘
```
<div dir="rtl">
<div dir="rtl" markdown="1">
قوانین مربوط به مرتب سازی ` Nan ` را در بخش [ORDER BY clause](../query_language/select.md#query_language-queries-order_by) ببینید.

View File

@ -1,6 +1,6 @@
<a name="data_types"></a>
<div dir="rtl">
<div dir="rtl" markdown="1">
# Data types
@ -8,4 +8,4 @@ ClickHouse قابلیت ذخیره سازی انواع type های مختلف ب
این بخش انواع data type های قابل پشتیبانی در ClickHouse را شرح می دهد، همچنین ملاحطات آنها در هنگام استفاده آنها را شرح می دهد.
</div>
</div>

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# UInt8, UInt16, UInt32, UInt64, Int8, Int16, Int32, Int64

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# AggregateFunction(name, types_of_arguments...)

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Nested data structures

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Nested(Name1 Type1, Name2 Type2, ...)
@ -31,7 +31,7 @@ CREATE TABLE test.visits
) ENGINE = CollapsingMergeTree(StartDate, intHash32(UserID), (CounterID, StartDate, intHash32(UserID), VisitID), 8192, Sign)
```
<div dir="rtl">
<div dir="rtl" markdown="1">
این مثال `Goals` را به عنوان یک ساختار داده nested تعریف می کند، که می تواند شامل داده های مربوط به conversion (اهداف رسیده) باشد. هر سطر در جدول `visit` می تواند با صفر یا چند coversion ارتباط داشته باشد.
@ -67,7 +67,7 @@ LIMIT 10
└────────────────────────────────┴───────────────────────────────────────────────────────────────────────────────────────────┘
```
<div dir="rtl">
<div dir="rtl" markdown="1">
ساده ترین راه برای فکر کردن به یک ساختار داده nestet این است که، یک nestet مجموعه ای از آرایه های چند ستونی با طول ثابت است.
@ -100,7 +100,7 @@ LIMIT 10
└─────────┴─────────────────────┘
```
<div dir="rtl">
<div dir="rtl" markdown="1">
شما نمیتوانید در قسمت SELECT تمام ساختار داده ی nested را قرار دهید. شما فقط می توانید ستون های فردی که هر کدام بخشی از این ساختار داده هستند را لیست کنید.

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Expression

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Special data types

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Set

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# String

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# Tuple(T1, T2, ...)

View File

@ -0,0 +1 @@
../../en/development/architecture.md

View File

@ -0,0 +1 @@
../../en/development/build.md

View File

@ -0,0 +1 @@
../../en/development/build_osx.md

View File

@ -0,0 +1 @@
../../en/development/index.md

View File

@ -0,0 +1 @@
../../en/development/style.md

View File

@ -0,0 +1 @@
../../en/development/tests.md

1
docs/fa/faq/general.md Symbolic link
View File

@ -0,0 +1 @@
../../en/faq/general.md

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# بنچمارک AMPLab Big Data
@ -23,7 +23,7 @@ s3cmd sync s3://big-data-benchmark/pavlo/text-deflate/5nodes/ .
cd ..
```
<div dir="rtl">
<div dir="rtl" markdown="1">
این query های ClickHouse را اجرا کنید:
@ -91,7 +91,7 @@ CREATE TABLE uservisits_5nodes_on_single
) ENGINE = MergeTree(visitDate, visitDate, 8192);
```
<div dir="rtl">
<div dir="rtl" markdown="1">
به کنسول برگردید و دستورات زیر را مجددا اجرا کنید:
@ -106,7 +106,7 @@ for i in 5nodes/rankings/*.deflate; do echo $i; zlib-flate -uncompress < $i | cl
for i in 5nodes/uservisits/*.deflate; do echo $i; zlib-flate -uncompress < $i | clickhouse-client --host=example-perftest01j --query="INSERT INTO uservisits_5nodes_on_single FORMAT CSV"; done
```
<div dir="rtl">
<div dir="rtl" markdown="1">
query های گرفتن data sample

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# ترابایت از لاگ های کلیک از سرویس Criteo
@ -12,7 +12,7 @@
CREATE TABLE criteo_log (date Date, clicked UInt8, int1 Int32, int2 Int32, int3 Int32, int4 Int32, int5 Int32, int6 Int32, int7 Int32, int8 Int32, int9 Int32, int10 Int32, int11 Int32, int12 Int32, int13 Int32, cat1 String, cat2 String, cat3 String, cat4 String, cat5 String, cat6 String, cat7 String, cat8 String, cat9 String, cat10 String, cat11 String, cat12 String, cat13 String, cat14 String, cat15 String, cat16 String, cat17 String, cat18 String, cat19 String, cat20 String, cat21 String, cat22 String, cat23 String, cat24 String, cat25 String, cat26 String) ENGINE = Log
```
<div dir="rtl">
<div dir="rtl" markdown="1">
داده ها را دانلود کنید:
@ -22,7 +22,7 @@ CREATE TABLE criteo_log (date Date, clicked UInt8, int1 Int32, int2 Int32, int3
for i in {00..23}; do echo $i; zcat datasets/criteo/day_${i#0}.gz | sed -r 's/^/2000-01-'${i/00/24}'\t/' | clickhouse-client --host=example-perftest01j --query="INSERT INTO criteo_log FORMAT TabSeparated"; done
```
<div dir="rtl">
<div dir="rtl" markdown="1">
یک جدول برای داده های تبدیل شده ایجاد کنید:
@ -75,7 +75,7 @@ CREATE TABLE criteo
) ENGINE = MergeTree(date, intHash32(icat1), (date, intHash32(icat1)), 8192)
```
<div dir="rtl">
<div dir="rtl" markdown="1">
داده ها را از لاگ raw انتقال و به جدول دوم وارد کنید:

File diff suppressed because one or more lines are too long

View File

@ -1,6 +1,6 @@
<a name="example_datasets-ontime"></a>
<div dir="rtl">
<div dir="rtl" markdown="1">
# OnTime
@ -18,7 +18,7 @@ done
done
```
<div dir="rtl">
<div dir="rtl" markdown="1">
(از <https://github.com/Percona-Lab/ontime-airline-performance/blob/master/download.sh> )
@ -140,7 +140,7 @@ CREATE TABLE `ontime` (
) ENGINE = MergeTree(FlightDate, (Year, FlightDate), 8192)
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Load داده ها:
@ -150,7 +150,7 @@ Load داده ها:
for i in *.zip; do echo $i; unzip -cq $i '*.csv' | sed 's/\.00//g' | clickhouse-client --host=example-perftest01j --query="INSERT INTO ontime FORMAT CSVWithNames"; done
```
<div dir="rtl">
<div dir="rtl" markdown="1">
query ها:
Q0.
@ -161,7 +161,7 @@ Q0.
select avg(c1) from (select Year, Month, count(*) as c1 from ontime group by Year, Month);
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q1. تعداد پروازهای به تفکیک روز از تاریخ 2000 تا 2008
@ -171,7 +171,7 @@ Q1. تعداد پروازهای به تفکیک روز از تاریخ 2000 تا
SELECT DayOfWeek, count(*) AS c FROM ontime WHERE Year >= 2000 AND Year <= 2008 GROUP BY DayOfWeek ORDER BY c DESC;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q2. تعداد پروازهای بیش از 10 دقیقه تاخیر خورده، گروه بندی براساس روزهای هفته از سال 2000 تا 2008
@ -181,7 +181,7 @@ Q2. تعداد پروازهای بیش از 10 دقیقه تاخیر خورده
SELECT DayOfWeek, count(*) AS c FROM ontime WHERE DepDelay>10 AND Year >= 2000 AND Year <= 2008 GROUP BY DayOfWeek ORDER BY c DESC
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q3. تعداد تاخیرها براساس airport از سال 2000 تا 2008
@ -191,7 +191,7 @@ Q3. تعداد تاخیرها براساس airport از سال 2000 تا 2008
SELECT Origin, count(*) AS c FROM ontime WHERE DepDelay>10 AND Year >= 2000 AND Year <= 2008 GROUP BY Origin ORDER BY c DESC LIMIT 10
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q4. تعداد تاخیرها براساس carrier در سال 78
@ -201,7 +201,7 @@ Q4. تعداد تاخیرها براساس carrier در سال 78
SELECT Carrier, count(*) FROM ontime WHERE DepDelay>10 AND Year = 2007 GROUP BY Carrier ORDER BY count(*) DESC
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q5. درصد تاخیر ها براساس carrier در سال 2007
@ -231,7 +231,7 @@ ANY INNER JOIN
ORDER BY c3 DESC;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
نسخه ی بهتر query
@ -241,7 +241,7 @@ ORDER BY c3 DESC;
SELECT Carrier, avg(DepDelay > 10) * 1000 AS c3 FROM ontime WHERE Year = 2007 GROUP BY Carrier ORDER BY Carrier
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q6. مانند query قبلی اما برای طیف وسیعی از سال های 2000 تا 2008
@ -271,7 +271,7 @@ ANY INNER JOIN
ORDER BY c3 DESC;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
نسخه ی بهتر query
@ -281,7 +281,7 @@ ORDER BY c3 DESC;
SELECT Carrier, avg(DepDelay > 10) * 1000 AS c3 FROM ontime WHERE Year >= 2000 AND Year <= 2008 GROUP BY Carrier ORDER BY Carrier
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q7. درصد تاخیر بیش از 10 دقیقه پروازها به تفکیک سال
@ -309,7 +309,7 @@ ANY INNER JOIN
ORDER BY Year
```
<div dir="rtl">
<div dir="rtl" markdown="1">
نسخه ی بهتر query
@ -319,7 +319,7 @@ ORDER BY Year
SELECT Year, avg(DepDelay > 10) FROM ontime GROUP BY Year ORDER BY Year
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q8. مقصدهای پرطرفدار براساس تعداد اتصال های مستقیم شهرها برای سال 2000 تا 2010
@ -329,7 +329,7 @@ Q8. مقصدهای پرطرفدار براساس تعداد اتصال های م
SELECT DestCityName, uniqExact(OriginCityName) AS u FROM ontime WHERE Year >= 2000 and Year <= 2010 GROUP BY DestCityName ORDER BY u DESC LIMIT 10;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q9.
@ -339,7 +339,7 @@ Q9.
select Year, count(*) as c1 from ontime group by Year;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
Q10.
@ -361,7 +361,7 @@ ORDER by rate DESC
LIMIT 1000;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
query های بیشتر:
@ -379,7 +379,7 @@ SELECT OriginCityName, DestCityName, count() AS c FROM ontime GROUP BY OriginCit
SELECT OriginCityName, count() AS c FROM ontime GROUP BY OriginCityName ORDER BY c DESC LIMIT 10;
```
<div dir="rtl">
<div dir="rtl" markdown="1">
این تست های performance توسط Vadim Tkachenko انجام شده است. برای اطلاعات بیشتر به لینک های زیر مراجعه کنید:

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# بنچمارک Star Schema
@ -12,7 +12,7 @@ cd ssb-dbgen
make
```
<div dir="rtl">
<div dir="rtl" markdown="1">
در هنگام پردازش چند warnings نمایش داده می شود که مشکلی نیست و طبیعی است.
@ -27,7 +27,7 @@ make
./dbgen -s 1000 -T l
```
<div dir="rtl">
<div dir="rtl" markdown="1">
ساخت جداول در ClickHouse
@ -84,7 +84,7 @@ CREATE TABLE customerd AS customer ENGINE = Distributed(perftest_3shards_1replic
CREATE TABLE partd AS part ENGINE = Distributed(perftest_3shards_1replicas, default, part, rand());
```
<div dir="rtl">
<div dir="rtl" markdown="1">
برای تست بر روی یک سرور، فقط از جداول MergeTree استفاده کنید. برای تست توزیع شده، شما نیاز به کانفیگ `perftest_3shards_1replicas` در فایل کانفیگ را دارید. در ادامه جداول MergeTree را در هر سرور ایجاد کنید و موارد بالا را توزیع کنید.

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# WikiStat
@ -22,7 +22,7 @@ CREATE TABLE wikistat
```
<div dir="rtl">
<div dir="rtl" markdown="1">
load دیتا

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# شروع به کار
@ -12,7 +12,7 @@
grep -q sse4_2 /proc/cpuinfo && echo "SSE 4.2 supported" || echo "SSE 4.2 not supported"
```
<div dir="rtl">
<div dir="rtl" markdown="1">
پیشنهاد می کنیم از Ubuntu TrustyT، Ubuntu Xenial یا Ubuntu Precise استفاده کنید. ترمینال باید از UTF-8 پشتیبانی کند. (به صورت پیش فرض در Ubuntu پشتیبانی می شود).
@ -30,7 +30,7 @@ grep -q sse4_2 /proc/cpuinfo && echo "SSE 4.2 supported" || echo "SSE 4.2 not su
deb http://repo.yandex.ru/clickhouse/deb/stable/ main/
```
<div dir="rtl">
<div dir="rtl" markdown="1">
اگر شما میخوایید جدیدترین نسخه ی تست را استفاده کنید، 'stable' رو به 'testing' تغییر بدید.
@ -44,7 +44,7 @@ sudo apt-get update
sudo apt-get install clickhouse-client clickhouse-server
```
<div dir="rtl">
<div dir="rtl" markdown="1">
شما همچنین می توانید از طریق لینک زیر پکیج ClickHouse را به صورت دستی دانلود و نصب کنید: <https://repo.yandex.ru/clickhouse/deb/stable/main/>.
@ -63,7 +63,7 @@ Client: dbms/programs/clickhouse-client
Server: dbms/programs/clickhouse-server
```
<div dir="rtl">
<div dir="rtl" markdown="1">
برای سرور، یک کاتالوگ با دیتا بسازید، مانند
@ -74,7 +74,7 @@ Server: dbms/programs/clickhouse-server
/opt/clickhouse/metadata/default/
```
<div dir="rtl">
<div dir="rtl" markdown="1">
(قابل تنظیم در تنظیمات سرور). 'chown' را برای کاربر دلخواه اجرا کنید.
@ -98,7 +98,7 @@ Gentoo: `emerge clickhouse`
sudo service clickhouse-server start
```
<div dir="rtl">
<div dir="rtl" markdown="1">
لاگ های دایرکتوری `/var/log/clickhouse-server/ directory.` را مشاهده کنید.
@ -112,7 +112,7 @@ sudo service clickhouse-server start
clickhouse-server --config-file=/etc/clickhouse-server/config.xml
```
<div dir="rtl">
<div dir="rtl" markdown="1">
در این مورد که مناسب زمان توسعه می باشد، لاگ ها در کنسول پرینت می شوند. اگر فایل تنظیمات در دایرکتوری جاری باشد، نیازی به مشخص کردن '--config-file' نمی باشد. به صورت پیش فرض از './config.xml' استفاده می شود.
@ -124,7 +124,7 @@ clickhouse-server --config-file=/etc/clickhouse-server/config.xml
clickhouse-client
```
<div dir="rtl">
<div dir="rtl" markdown="1">
پارامترهای پیش فرض، نشان از اتصال به localhost:9000 از طرف کاربر 'default' بدون پسورد را می دهد. از کلاینت میتوان برای اتصال به یک سرور remote استفاده کرد. مثال:
@ -134,7 +134,7 @@ clickhouse-client
clickhouse-client --host=example.com
```
<div dir="rtl">
<div dir="rtl" markdown="1">
برای اطلاعات بیشتر، بخش "کلاینت Command-line" را مشاهده کنید.
@ -161,7 +161,7 @@ SELECT 1
:)
```
<div dir="rtl">
<div dir="rtl" markdown="1">
**تبریک میگم، سیستم کار می کنه!**

Binary file not shown.

After

Width:  |  Height:  |  Size: 44 KiB

12
docs/fa/images/logo.svg Normal file
View File

@ -0,0 +1,12 @@
<svg xmlns="http://www.w3.org/2000/svg" width="54" height="48" viewBox="0 0 9 8">
<style>
.o{fill:#fc0}
.r{fill:#f00}
</style>
<path class="r" d="M0,7 h1 v1 h-1 z"/>
<path class="o" d="M0,0 h1 v7 h-1 z"/>
<path class="o" d="M2,0 h1 v8 h-1 z"/>
<path class="o" d="M4,0 h1 v8 h-1 z"/>
<path class="o" d="M6,0 h1 v8 h-1 z"/>
<path class="o" d="M8,3.25 h1 v1.5 h-1 z"/>
</svg>

After

Width:  |  Height:  |  Size: 421 B

Binary file not shown.

After

Width:  |  Height:  |  Size: 41 KiB

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# ClickHouse چیست؟

View File

@ -1,4 +1,4 @@
<div dir="rtl">
<div dir="rtl" markdown="1">
# کلاینت Command-line
@ -15,7 +15,7 @@ Connected to ClickHouse server version 0.0.26176.
:)
```
<div dir="rtl">
<div dir="rtl" markdown="1">
کلاینت از آپشن های command-line و فایل های کانفیگ پشتیبانی می کند. برای اطلاعات بیشتر بخش "[پیکربندی](#interfaces_cli_configuration)" را مشاهده کنید.
@ -37,7 +37,7 @@ _EOF
cat file.csv | clickhouse-client --database=test --query="INSERT INTO test FORMAT CSV";
```
<div dir="rtl">
<div dir="rtl" markdown="1">
در حالت Batch، فرمت داده ها به صورت پیش فرض به صورت TabSeparated می باشد. شما میتوانید فرمت داده ها رو در هنگام اجرای query و با استفاده از شرط FORMAT مشخص کنید.

Some files were not shown because too many files have changed in this diff Show More