This commit is contained in:
Roman Peshkurov 2015-06-16 19:22:13 +03:00
commit 2f606be303
117 changed files with 1325 additions and 524 deletions

View File

@ -1,90 +0,0 @@
#include <DB/AggregateFunctions/IUnaryAggregateFunction.h>
namespace DB
{
/** Сделано в целях отладки. Подлежит удалению.
*/
struct AggregateFunctionDebugData
{
UInt32 value;
AggregateFunctionDebugData()
{
value = 0xAAAAAAAA;
if (rand() % 1000 == 0)
throw Exception("Test1");
}
~AggregateFunctionDebugData()
{
try
{
if (value == 0xDEADDEAD)
throw Exception("Double free");
if (value != 0xAAAAAAAA)
throw Exception("Corruption");
}
catch (...)
{
tryLogCurrentException(__PRETTY_FUNCTION__);
std::terminate();
}
value = 0xDEADDEAD;
}
};
class AggregateFunctionDebug final : public IUnaryAggregateFunction<AggregateFunctionDebugData, AggregateFunctionDebug>
{
public:
String getName() const { return "debug"; }
DataTypePtr getReturnType() const
{
return new DataTypeUInt32;
}
void setArgument(const DataTypePtr & argument)
{
}
void addOne(AggregateDataPtr place, const IColumn & column, size_t row_num) const
{
if (rand() % 1000 == 0)
throw Exception("Test2");
}
void merge(AggregateDataPtr place, ConstAggregateDataPtr rhs) const
{
if (rand() % 1000 == 0)
throw Exception("Test3");
}
void serialize(ConstAggregateDataPtr place, WriteBuffer & buf) const
{
if (rand() % 1000 == 0)
throw Exception("Test4");
}
void deserializeMerge(AggregateDataPtr place, ReadBuffer & buf) const
{
if (rand() % 1000 == 0)
throw Exception("Test5");
}
void insertResultInto(ConstAggregateDataPtr place, IColumn & to) const
{
if (rand() % 1000 == 0)
throw Exception("Test6");
static_cast<ColumnUInt32 &>(to).getData().push_back(123);
}
};
}

View File

@ -151,20 +151,18 @@ public:
void * realloc(void * buf, size_t old_size, size_t new_size) void * realloc(void * buf, size_t old_size, size_t new_size)
{ {
/// Было в stack_memory, там и останется.
if (new_size <= N) if (new_size <= N)
return buf; return buf;
/// Уже не помещалось в stack_memory.
if (old_size > N) if (old_size > N)
return HashTableAllocator::realloc(buf, old_size, new_size); return HashTableAllocator::realloc(buf, old_size, new_size);
buf = ::malloc(new_size); /// Было в stack_memory, но теперь не помещается.
if (nullptr == buf) void * new_buf = HashTableAllocator::alloc(new_size);
DB::throwFromErrno("HashTableAllocator: Cannot malloc.", DB::ErrorCodes::CANNOT_ALLOCATE_MEMORY); memcpy(new_buf, buf, old_size);
return new_buf;
memcpy(buf, stack_memory, old_size);
memset(reinterpret_cast<char *>(buf) + old_size, 0, new_size - old_size);
return buf;
} }
}; };

View File

@ -51,6 +51,7 @@
M(DelayedInserts) \ M(DelayedInserts) \
M(RejectedInserts) \ M(RejectedInserts) \
M(DelayedInsertsMilliseconds) \ M(DelayedInsertsMilliseconds) \
M(SynchronousMergeOnInsert) \
\ \
M(ZooKeeperInit) \ M(ZooKeeperInit) \
M(ZooKeeperTransactions) \ M(ZooKeeperTransactions) \

View File

@ -76,6 +76,7 @@ public:
v3 = 0x7465646279746573ULL ^ k1; v3 = 0x7465646279746573ULL ^ k1;
cnt = 0; cnt = 0;
current_word = 0;
} }
void update(const char * data, u64 size) void update(const char * data, u64 size)

View File

@ -25,7 +25,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "AddingConstColumnBlockInputStream"; } String getName() const override { return "AddingConstColumn"; }
String getID() const override String getID() const override
{ {

View File

@ -35,7 +35,7 @@ public:
{ {
} }
String getName() const override { return "AddingDefaultBlockInputStream"; } String getName() const override { return "AddingDefault"; }
String getID() const override String getID() const override
{ {

View File

@ -32,7 +32,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "AggregatingBlockInputStream"; } String getName() const override { return "Aggregating"; }
String getID() const override String getID() const override
{ {

View File

@ -26,7 +26,7 @@ public:
{ {
} }
String getName() const override { return "AggregatingSortedBlockInputStream"; } String getName() const override { return "AggregatingSorted"; }
String getID() const override String getID() const override
{ {

View File

@ -25,7 +25,7 @@ public:
children.push_back(in_); children.push_back(in_);
} }
String getName() const override { return "AsynchronousBlockInputStream"; } String getName() const override { return "Asynchronous"; }
String getID() const override String getID() const override
{ {

View File

@ -29,8 +29,6 @@ struct BlockStreamProfileInfo
using BlockStreamProfileInfos = std::vector<const BlockStreamProfileInfo *>; using BlockStreamProfileInfos = std::vector<const BlockStreamProfileInfo *>;
BlockStreamProfileInfos nested_infos; BlockStreamProfileInfos nested_infos;
String column_names;
/// Собрать BlockStreamProfileInfo для ближайших в дереве источников с именем name. Пример; собрать все info для PartialSorting stream-ов. /// Собрать BlockStreamProfileInfo для ближайших в дереве источников с именем name. Пример; собрать все info для PartialSorting stream-ов.
void collectInfosForStreamsWithName(const char * name, BlockStreamProfileInfos & res) const; void collectInfosForStreamsWithName(const char * name, BlockStreamProfileInfos & res) const;

View File

@ -23,7 +23,7 @@ public:
~CollapsingFinalBlockInputStream(); ~CollapsingFinalBlockInputStream();
String getName() const override { return "CollapsingFinalBlockInputStream"; } String getName() const override { return "CollapsingFinal"; }
String getID() const override String getID() const override
{ {

View File

@ -30,7 +30,7 @@ public:
{ {
} }
String getName() const override { return "CollapsingSortedBlockInputStream"; } String getName() const override { return "CollapsingSorted"; }
String getID() const override String getID() const override
{ {

View File

@ -22,7 +22,7 @@ public:
current_stream = children.begin(); current_stream = children.begin();
} }
String getName() const override { return "ConcatBlockInputStream"; } String getName() const override { return "Concat"; }
String getID() const override String getID() const override
{ {

View File

@ -28,7 +28,7 @@ public:
children.push_back(input); children.push_back(input);
} }
String getName() const override { return "CreatingSetsBlockInputStream"; } String getName() const override { return "CreatingSets"; }
String getID() const override String getID() const override
{ {

View File

@ -20,7 +20,7 @@ public:
/// Пустой columns_ значит все столбцы. /// Пустой columns_ значит все столбцы.
DistinctBlockInputStream(BlockInputStreamPtr input_, const Limits & limits, size_t limit_, Names columns_); DistinctBlockInputStream(BlockInputStreamPtr input_, const Limits & limits, size_t limit_, Names columns_);
String getName() const override { return "DistinctBlockInputStream"; } String getName() const override { return "Distinct"; }
String getID() const override String getID() const override
{ {

View File

@ -26,7 +26,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "ExpressionBlockInputStream"; } String getName() const override { return "Expression"; }
String getID() const override String getID() const override
{ {

View File

@ -22,7 +22,7 @@ public:
FilterBlockInputStream(BlockInputStreamPtr input_, ssize_t filter_column_); FilterBlockInputStream(BlockInputStreamPtr input_, ssize_t filter_column_);
FilterBlockInputStream(BlockInputStreamPtr input_, const String & filter_column_name_); FilterBlockInputStream(BlockInputStreamPtr input_, const String & filter_column_name_);
String getName() const override { return "FilterBlockInputStream"; } String getName() const override { return "Filter"; }
String getID() const override String getID() const override
{ {

View File

@ -51,7 +51,6 @@ public:
/** Для вывода дерева преобразований потока данных (плана выполнения запроса). /** Для вывода дерева преобразований потока данных (плана выполнения запроса).
*/ */
virtual String getName() const = 0; virtual String getName() const = 0;
virtual String getShortName() const; /// То же самое, но без BlockInputStream на конце.
/** Уникальный идентификатор части конвейера выполнения запроса. /** Уникальный идентификатор части конвейера выполнения запроса.
* Источники с одинаковым идентификатором считаются идентичными * Источники с одинаковым идентификатором считаются идентичными

View File

@ -18,7 +18,7 @@ public:
LazyBlockInputStream(Generator generator_) LazyBlockInputStream(Generator generator_)
: generator(generator_) {} : generator(generator_) {}
String getName() const override { return "LazyBlockInputStream"; } String getName() const override { return "Lazy"; }
String getID() const override String getID() const override
{ {

View File

@ -18,7 +18,7 @@ class LimitBlockInputStream : public IProfilingBlockInputStream
public: public:
LimitBlockInputStream(BlockInputStreamPtr input_, size_t limit_, size_t offset_ = 0); LimitBlockInputStream(BlockInputStreamPtr input_, size_t limit_, size_t offset_ = 0);
String getName() const override { return "LimitBlockInputStream"; } String getName() const override { return "Limit"; }
String getID() const override String getID() const override
{ {

View File

@ -18,7 +18,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "MaterializingBlockInputStream"; } String getName() const override { return "Materializing"; }
String getID() const override String getID() const override
{ {

View File

@ -31,7 +31,7 @@ public:
MergeSortingBlocksBlockInputStream(Blocks & blocks_, SortDescription & description_, MergeSortingBlocksBlockInputStream(Blocks & blocks_, SortDescription & description_,
size_t max_merged_block_size_, size_t limit_ = 0); size_t max_merged_block_size_, size_t limit_ = 0);
String getName() const override { return "MergeSortingBlocksBlockInputStream"; } String getName() const override { return "MergeSortingBlocks"; }
String getID() const override { return getName(); } String getID() const override { return getName(); }
protected: protected:
@ -73,7 +73,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "MergeSortingBlockInputStream"; } String getName() const override { return "MergeSorting"; }
String getID() const override String getID() const override
{ {

View File

@ -24,7 +24,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "MergingAggregatedBlockInputStream"; } String getName() const override { return "MergingAggregated"; }
String getID() const override String getID() const override
{ {

View File

@ -26,7 +26,7 @@ public:
children.insert(children.end(), inputs_.begin(), inputs_.end()); children.insert(children.end(), inputs_.begin(), inputs_.end());
} }
String getName() const override { return "MergingSortedBlockInputStream"; } String getName() const override { return "MergingSorted"; }
String getID() const override String getID() const override
{ {

View File

@ -18,7 +18,7 @@ public:
NativeBlockInputStream(ReadBuffer & istr_, UInt64 server_revision_ = 0) NativeBlockInputStream(ReadBuffer & istr_, UInt64 server_revision_ = 0)
: istr(istr_), server_revision(server_revision_) {} : istr(istr_), server_revision(server_revision_) {}
String getName() const override { return "NativeBlockInputStream"; } String getName() const override { return "Native"; }
String getID() const override String getID() const override
{ {

View File

@ -22,7 +22,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "NullAndDoCopyBlockInputStream"; } String getName() const override { return "NullAndDoCopy"; }
String getID() const override String getID() const override
{ {

View File

@ -12,7 +12,7 @@ class NullBlockInputStream : public IBlockInputStream
{ {
public: public:
Block read() override { return Block(); } Block read() override { return Block(); }
String getName() const override { return "NullBlockInputStream"; } String getName() const override { return "Null"; }
String getID() const override String getID() const override
{ {

View File

@ -18,7 +18,7 @@ class OneBlockInputStream : public IProfilingBlockInputStream
public: public:
OneBlockInputStream(const Block & block_) : block(block_) {} OneBlockInputStream(const Block & block_) : block(block_) {}
String getName() const override { return "OneBlockInputStream"; } String getName() const override { return "One"; }
String getID() const override String getID() const override
{ {

View File

@ -34,7 +34,7 @@ public:
children.insert(children.end(), inputs.begin(), inputs.end()); children.insert(children.end(), inputs.begin(), inputs.end());
} }
String getName() const override { return "ParallelAggregatingBlockInputStream"; } String getName() const override { return "ParallelAggregating"; }
String getID() const override String getID() const override
{ {

View File

@ -21,7 +21,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "PartialSortingBlockInputStream"; } String getName() const override { return "PartialSorting"; }
String getID() const override String getID() const override
{ {

View File

@ -61,7 +61,7 @@ public:
} }
String getName() const override { return "RemoteBlockInputStream"; } String getName() const override { return "Remote"; }
String getID() const override String getID() const override

View File

@ -24,7 +24,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "RemoveColumnsBlockInputStream"; } String getName() const override { return "RemoveColumns"; }
String getID() const override String getID() const override
{ {

View File

@ -30,7 +30,7 @@ public:
{ {
} }
String getName() const override { return "SummingSortedBlockInputStream"; } String getName() const override { return "SummingSorted"; }
String getID() const override String getID() const override
{ {

View File

@ -28,7 +28,7 @@ public:
children.push_back(input_); children.push_back(input_);
} }
String getName() const override { return "TotalsHavingBlockInputStream"; } String getName() const override { return "TotalsHaving"; }
String getID() const override String getID() const override
{ {

View File

@ -36,7 +36,7 @@ public:
children = inputs; children = inputs;
} }
String getName() const override { return "UnionBlockInputStream"; } String getName() const override { return "Union"; }
String getID() const override String getID() const override
{ {

View File

@ -13,7 +13,7 @@ using Poco::SharedPtr;
/** Тип - состояние агрегатной функции. /** Тип - состояние агрегатной функции.
* Параметры типа - это агрегатная функция, типы её аргументов и её параметры (для параметрических агрегатных функций). * Параметры типа - это агрегатная функция, типы её аргументов и её параметры (для параметрических агрегатных функций).
*/ */
class DataTypeAggregateFunction : public IDataType class DataTypeAggregateFunction final : public IDataType
{ {
private: private:
AggregateFunctionPtr function; AggregateFunctionPtr function;

View File

@ -9,7 +9,7 @@ namespace DB
using Poco::SharedPtr; using Poco::SharedPtr;
class DataTypeArray : public IDataType class DataTypeArray final : public IDataType
{ {
private: private:
/// Тип элементов массивов. /// Тип элементов массивов.

View File

@ -10,7 +10,7 @@
namespace DB namespace DB
{ {
class DataTypeDate : public IDataTypeNumberFixed<UInt16, ColumnUInt16> class DataTypeDate final : public IDataTypeNumberFixed<UInt16, ColumnUInt16>
{ {
public: public:
DataTypeDate() {} DataTypeDate() {}

View File

@ -10,7 +10,7 @@
namespace DB namespace DB
{ {
class DataTypeDateTime : public IDataTypeNumberFixed<UInt32, ColumnUInt32> class DataTypeDateTime final : public IDataTypeNumberFixed<UInt32, ColumnUInt32>
{ {
public: public:
DataTypeDateTime() {} DataTypeDateTime() {}

View File

@ -9,7 +9,7 @@ namespace DB
/** /**
* Лямбда-выражение. * Лямбда-выражение.
*/ */
class DataTypeExpression : public IDataTypeDummy class DataTypeExpression final : public IDataTypeDummy
{ {
private: private:
DataTypes argument_types; DataTypes argument_types;

View File

@ -13,7 +13,7 @@ namespace DB
using Poco::SharedPtr; using Poco::SharedPtr;
class DataTypeFixedString : public IDataType class DataTypeFixedString final : public IDataType
{ {
private: private:
size_t n; size_t n;

View File

@ -9,7 +9,7 @@ namespace DB
using Poco::SharedPtr; using Poco::SharedPtr;
class DataTypeNested : public IDataType class DataTypeNested final : public IDataType
{ {
private: private:
/// Имена и типы вложенных массивов. /// Имена и типы вложенных массивов.

View File

@ -9,7 +9,7 @@ namespace DB
/** Тип данных, соответствующий множеству значений в секции IN. /** Тип данных, соответствующий множеству значений в секции IN.
* Используется только как промежуточный вариант при вычислении выражений. * Используется только как промежуточный вариант при вычислении выражений.
*/ */
class DataTypeSet : public IDataTypeDummy class DataTypeSet final : public IDataTypeDummy
{ {
public: public:
std::string getName() const { return "Set"; } std::string getName() const { return "Set"; }

View File

@ -13,7 +13,7 @@ namespace DB
using Poco::SharedPtr; using Poco::SharedPtr;
class DataTypeString : public IDataType class DataTypeString final : public IDataType
{ {
public: public:
using FieldType = String; using FieldType = String;

View File

@ -15,7 +15,7 @@ namespace DB
* Также может быть использовать в качестве столбца - результата выполнения запроса. * Также может быть использовать в качестве столбца - результата выполнения запроса.
* Не может быть сохранён в таблицы. * Не может быть сохранён в таблицы.
*/ */
class DataTypeTuple : public IDataType class DataTypeTuple final : public IDataType
{ {
private: private:
DataTypes elems; DataTypes elems;
@ -138,6 +138,7 @@ public:
{ {
ColumnWithNameAndType col; ColumnWithNameAndType col;
col.column = elems[i]->createColumn(); col.column = elems[i]->createColumn();
col.type = elems[i]->clone();
tuple_block.insert(col); tuple_block.insert(col);
} }
return new ColumnTuple(tuple_block); return new ColumnTuple(tuple_block);

View File

@ -13,11 +13,11 @@ template <typename T>
struct DataTypeFromFieldType; struct DataTypeFromFieldType;
#define DEFINE_DATA_TYPE_NUMBER_FIXED(TYPE) \ #define DEFINE_DATA_TYPE_NUMBER_FIXED(TYPE) \
class DataType ## TYPE : public IDataTypeNumberFixed<TYPE, Column ## TYPE> \ class DataType ## TYPE final : public IDataTypeNumberFixed<TYPE, Column ## TYPE> \
{ \ { \
public: \ public: \
std::string getName() const { return #TYPE; } \ std::string getName() const { return #TYPE; } \
DataTypePtr clone() const { return new DataType ## TYPE; } \ DataTypePtr clone() const { return new DataType ## TYPE; } \
}; \ }; \
\ \
template <> struct DataTypeFromFieldType<TYPE> \ template <> struct DataTypeFromFieldType<TYPE> \

View File

@ -42,6 +42,8 @@ public:
: CacheDictionary{other.name, other.dict_struct, other.source_ptr->clone(), other.dict_lifetime, other.size} : CacheDictionary{other.name, other.dict_struct, other.source_ptr->clone(), other.dict_lifetime, other.size}
{} {}
std::exception_ptr getCreationException() const override { return {}; }
std::string getName() const override { return name; } std::string getName() const override { return name; }
std::string getTypeName() const override { return "Cache"; } std::string getTypeName() const override { return "Cache"; }

View File

@ -25,8 +25,17 @@ public:
source_ptr{std::move(source_ptr)}, dict_lifetime(dict_lifetime) source_ptr{std::move(source_ptr)}, dict_lifetime(dict_lifetime)
{ {
createAttributes(); createAttributes();
loadData();
calculateBytesAllocated(); try
{
loadData();
calculateBytesAllocated();
}
catch (...)
{
creation_exception = std::current_exception();
}
creation_time = std::chrono::system_clock::now(); creation_time = std::chrono::system_clock::now();
} }
@ -34,6 +43,8 @@ public:
: FlatDictionary{other.name, other.dict_struct, other.source_ptr->clone(), other.dict_lifetime} : FlatDictionary{other.name, other.dict_struct, other.source_ptr->clone(), other.dict_lifetime}
{} {}
std::exception_ptr getCreationException() const override { return creation_exception; }
std::string getName() const override { return name; } std::string getName() const override { return name; }
std::string getTypeName() const override { return "Flat"; } std::string getTypeName() const override { return "Flat"; }
@ -398,10 +409,11 @@ private:
std::size_t bytes_allocated = 0; std::size_t bytes_allocated = 0;
std::size_t element_count = 0; std::size_t element_count = 0;
std::size_t bucket_count = 0; std::size_t bucket_count = 0;
mutable std::atomic<std::size_t> query_count;
std::chrono::time_point<std::chrono::system_clock> creation_time; std::chrono::time_point<std::chrono::system_clock> creation_time;
mutable std::atomic<std::size_t> query_count; std::exception_ptr creation_exception;
}; };
} }

View File

@ -22,8 +22,17 @@ public:
source_ptr{std::move(source_ptr)}, dict_lifetime(dict_lifetime) source_ptr{std::move(source_ptr)}, dict_lifetime(dict_lifetime)
{ {
createAttributes(); createAttributes();
loadData();
calculateBytesAllocated(); try
{
loadData();
calculateBytesAllocated();
}
catch (...)
{
creation_exception = std::current_exception();
}
creation_time = std::chrono::system_clock::now(); creation_time = std::chrono::system_clock::now();
} }
@ -31,6 +40,8 @@ public:
: HashedDictionary{other.name, other.dict_struct, other.source_ptr->clone(), other.dict_lifetime} : HashedDictionary{other.name, other.dict_struct, other.source_ptr->clone(), other.dict_lifetime}
{} {}
std::exception_ptr getCreationException() const override { return creation_exception; }
std::string getName() const override { return name; } std::string getName() const override { return name; }
std::string getTypeName() const override { return "Hashed"; } std::string getTypeName() const override { return "Hashed"; }
@ -389,6 +400,8 @@ private:
mutable std::atomic<std::size_t> query_count{}; mutable std::atomic<std::size_t> query_count{};
std::chrono::time_point<std::chrono::system_clock> creation_time; std::chrono::time_point<std::chrono::system_clock> creation_time;
std::exception_ptr creation_exception;
}; };
} }

View File

@ -24,6 +24,8 @@ class IDictionary
public: public:
using id_t = std::uint64_t; using id_t = std::uint64_t;
virtual std::exception_ptr getCreationException() const = 0;
virtual std::string getName() const = 0; virtual std::string getName() const = 0;
virtual std::string getTypeName() const = 0; virtual std::string getTypeName() const = 0;

View File

@ -90,7 +90,7 @@ public:
} }
} }
String getName() const override { return "MySQLBlockInputStream"; } String getName() const override { return "MySQL"; }
String getID() const override String getID() const override
{ {

View File

@ -22,7 +22,7 @@ public:
private: private:
Block readImpl() override { return stream->read(); } Block readImpl() override { return stream->read(); }
String getName() const override { return "OwningBufferBlockInputStream"; } String getName() const override { return "OwningBuffer"; }
String getID() const override { return "OwningBuffer(" + stream->getID() + ")"; } String getID() const override { return "OwningBuffer(" + stream->getID() + ")"; }

View File

@ -26,6 +26,10 @@ namespace DB
* - даты; * - даты;
* - даты-с-временем; * - даты-с-временем;
* внутри каждой группы, но не из разных групп. * внутри каждой группы, но не из разных групп.
*
* Исключение: можно сравнивать дату и дату-с-временем с константной строкой. Пример: EventDate = '2015-01-01'.
*
* TODO Массивы, кортежи.
*/ */
/** Игнорируем warning о сравнении signed и unsigned. /** Игнорируем warning о сравнении signed и unsigned.
@ -391,9 +395,9 @@ public:
private: private:
template <typename T0, typename T1> template <typename T0, typename T1>
bool executeNumRightType(Block & block, const ColumnNumbers & arguments, size_t result, const ColumnVector<T0> * col_left) bool executeNumRightType(Block & block, size_t result, const ColumnVector<T0> * col_left, const IColumn * col_right_untyped)
{ {
if (ColumnVector<T1> * col_right = typeid_cast<ColumnVector<T1> *>(&*block.getByPosition(arguments[1]).column)) if (const ColumnVector<T1> * col_right = typeid_cast<const ColumnVector<T1> *>(col_right_untyped))
{ {
ColumnUInt8 * col_res = new ColumnUInt8; ColumnUInt8 * col_res = new ColumnUInt8;
block.getByPosition(result).column = col_res; block.getByPosition(result).column = col_res;
@ -404,7 +408,7 @@ private:
return true; return true;
} }
else if (ColumnConst<T1> * col_right = typeid_cast<ColumnConst<T1> *>(&*block.getByPosition(arguments[1]).column)) else if (const ColumnConst<T1> * col_right = typeid_cast<const ColumnConst<T1> *>(col_right_untyped))
{ {
ColumnUInt8 * col_res = new ColumnUInt8; ColumnUInt8 * col_res = new ColumnUInt8;
block.getByPosition(result).column = col_res; block.getByPosition(result).column = col_res;
@ -420,9 +424,9 @@ private:
} }
template <typename T0, typename T1> template <typename T0, typename T1>
bool executeNumConstRightType(Block & block, const ColumnNumbers & arguments, size_t result, const ColumnConst<T0> * col_left) bool executeNumConstRightType(Block & block, size_t result, const ColumnConst<T0> * col_left, const IColumn * col_right_untyped)
{ {
if (ColumnVector<T1> * col_right = typeid_cast<ColumnVector<T1> *>(&*block.getByPosition(arguments[1]).column)) if (const ColumnVector<T1> * col_right = typeid_cast<const ColumnVector<T1> *>(col_right_untyped))
{ {
ColumnUInt8 * col_res = new ColumnUInt8; ColumnUInt8 * col_res = new ColumnUInt8;
block.getByPosition(result).column = col_res; block.getByPosition(result).column = col_res;
@ -433,7 +437,7 @@ private:
return true; return true;
} }
else if (ColumnConst<T1> * col_right = typeid_cast<ColumnConst<T1> *>(&*block.getByPosition(arguments[1]).column)) else if (const ColumnConst<T1> * col_right = typeid_cast<const ColumnConst<T1> *>(col_right_untyped))
{ {
UInt8 res = 0; UInt8 res = 0;
NumComparisonImpl<T0, T1, Op<T0, T1>>::constant_constant(col_left->getData(), col_right->getData(), res); NumComparisonImpl<T0, T1, Op<T0, T1>>::constant_constant(col_left->getData(), col_right->getData(), res);
@ -448,41 +452,41 @@ private:
} }
template <typename T0> template <typename T0>
bool executeNumLeftType(Block & block, const ColumnNumbers & arguments, size_t result) bool executeNumLeftType(Block & block, size_t result, const IColumn * col_left_untyped, const IColumn * col_right_untyped)
{ {
if (ColumnVector<T0> * col_left = typeid_cast<ColumnVector<T0> *>(&*block.getByPosition(arguments[0]).column)) if (const ColumnVector<T0> * col_left = typeid_cast<const ColumnVector<T0> *>(col_left_untyped))
{ {
if ( executeNumRightType<T0, UInt8>(block, arguments, result, col_left) if ( executeNumRightType<T0, UInt8>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, UInt16>(block, arguments, result, col_left) || executeNumRightType<T0, UInt16>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, UInt32>(block, arguments, result, col_left) || executeNumRightType<T0, UInt32>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, UInt64>(block, arguments, result, col_left) || executeNumRightType<T0, UInt64>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, Int8>(block, arguments, result, col_left) || executeNumRightType<T0, Int8>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, Int16>(block, arguments, result, col_left) || executeNumRightType<T0, Int16>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, Int32>(block, arguments, result, col_left) || executeNumRightType<T0, Int32>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, Int64>(block, arguments, result, col_left) || executeNumRightType<T0, Int64>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, Float32>(block, arguments, result, col_left) || executeNumRightType<T0, Float32>(block, result, col_left, col_right_untyped)
|| executeNumRightType<T0, Float64>(block, arguments, result, col_left)) || executeNumRightType<T0, Float64>(block, result, col_left, col_right_untyped))
return true; return true;
else else
throw Exception("Illegal column " + block.getByPosition(arguments[1]).column->getName() throw Exception("Illegal column " + col_right_untyped->getName()
+ " of second argument of function " + getName(), + " of second argument of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN); ErrorCodes::ILLEGAL_COLUMN);
} }
else if (ColumnConst<T0> * col_left = typeid_cast<ColumnConst<T0> *>(&*block.getByPosition(arguments[0]).column)) else if (const ColumnConst<T0> * col_left = typeid_cast<const ColumnConst<T0> *>(col_left_untyped))
{ {
if ( executeNumConstRightType<T0, UInt8>(block, arguments, result, col_left) if ( executeNumConstRightType<T0, UInt8>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, UInt16>(block, arguments, result, col_left) || executeNumConstRightType<T0, UInt16>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, UInt32>(block, arguments, result, col_left) || executeNumConstRightType<T0, UInt32>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, UInt64>(block, arguments, result, col_left) || executeNumConstRightType<T0, UInt64>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, Int8>(block, arguments, result, col_left) || executeNumConstRightType<T0, Int8>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, Int16>(block, arguments, result, col_left) || executeNumConstRightType<T0, Int16>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, Int32>(block, arguments, result, col_left) || executeNumConstRightType<T0, Int32>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, Int64>(block, arguments, result, col_left) || executeNumConstRightType<T0, Int64>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, Float32>(block, arguments, result, col_left) || executeNumConstRightType<T0, Float32>(block, result, col_left, col_right_untyped)
|| executeNumConstRightType<T0, Float64>(block, arguments, result, col_left)) || executeNumConstRightType<T0, Float64>(block, result, col_left, col_right_untyped))
return true; return true;
else else
throw Exception("Illegal column " + block.getByPosition(arguments[1]).column->getName() throw Exception("Illegal column " + col_right_untyped->getName()
+ " of second argument of function " + getName(), + " of second argument of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN); ErrorCodes::ILLEGAL_COLUMN);
} }
@ -490,17 +494,14 @@ private:
return false; return false;
} }
void executeString(Block & block, const ColumnNumbers & arguments, size_t result) void executeString(Block & block, size_t result, const IColumn * c0, const IColumn * c1)
{ {
IColumn * c0 = &*block.getByPosition(arguments[0]).column; const ColumnString * c0_string = typeid_cast<const ColumnString *>(c0);
IColumn * c1 = &*block.getByPosition(arguments[1]).column; const ColumnString * c1_string = typeid_cast<const ColumnString *>(c1);
const ColumnFixedString * c0_fixed_string = typeid_cast<const ColumnFixedString *>(c0);
ColumnString * c0_string = typeid_cast<ColumnString *>(c0); const ColumnFixedString * c1_fixed_string = typeid_cast<const ColumnFixedString *>(c1);
ColumnString * c1_string = typeid_cast<ColumnString *>(c1); const ColumnConstString * c0_const = typeid_cast<const ColumnConstString *>(c0);
ColumnFixedString * c0_fixed_string = typeid_cast<ColumnFixedString *>(c0); const ColumnConstString * c1_const = typeid_cast<const ColumnConstString *>(c1);
ColumnFixedString * c1_fixed_string = typeid_cast<ColumnFixedString *>(c1);
ColumnConstString * c0_const = typeid_cast<ColumnConstString *>(c0);
ColumnConstString * c1_const = typeid_cast<ColumnConstString *>(c1);
using StringImpl = StringComparisonImpl<Op<int, int>>; using StringImpl = StringComparisonImpl<Op<int, int>>;
@ -559,13 +560,66 @@ private:
c_res->getData()); c_res->getData());
else else
throw Exception("Illegal columns " throw Exception("Illegal columns "
+ block.getByPosition(arguments[0]).column->getName() + " and " + c0->getName() + " and " + c1->getName()
+ block.getByPosition(arguments[1]).column->getName()
+ " of arguments of function " + getName(), + " of arguments of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN); ErrorCodes::ILLEGAL_COLUMN);
} }
} }
void executeDateOrDateTimeWithConstString(Block & block, size_t result,
const IColumn * col_left_untyped, const IColumn * col_right_untyped,
bool left_is_num, bool right_is_num)
{
/// Особый случай - сравнение дат и дат-с-временем со строковой константой.
const IColumn * column_date_or_datetime = left_is_num ? col_left_untyped : col_right_untyped;
const IColumn * column_string_untyped = !left_is_num ? col_left_untyped : col_right_untyped;
bool is_date = false;
bool is_date_time = false;
is_date = typeid_cast<const ColumnVector<DataTypeDate::FieldType> *>(column_date_or_datetime)
|| typeid_cast<const ColumnConst<DataTypeDate::FieldType> *>(column_date_or_datetime);
if (!is_date)
is_date_time = typeid_cast<const ColumnVector<DataTypeDateTime::FieldType> *>(column_date_or_datetime)
|| typeid_cast<const ColumnConst<DataTypeDateTime::FieldType> *>(column_date_or_datetime);
const ColumnConstString * column_string = typeid_cast<const ColumnConstString *>(column_string_untyped);
if (!column_string
|| (!is_date && !is_date_time))
throw Exception("Illegal columns " + col_left_untyped->getName() + " and " + col_right_untyped->getName()
+ " of arguments of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN);
if (is_date)
{
DayNum_t date;
ReadBufferFromString in(column_string->getData());
readDateText(date, in);
if (!in.eof())
throw Exception("String is too long for Date: " + column_string->getData());
ColumnConst<DataTypeDate::FieldType> parsed_const_date(block.rowsInFirstColumn(), date);
executeNumLeftType<DataTypeDate::FieldType>(block, result,
left_is_num ? col_left_untyped : &parsed_const_date,
left_is_num ? &parsed_const_date : col_right_untyped);
}
else if (is_date_time)
{
time_t date_time;
ReadBufferFromString in(column_string->getData());
readDateTimeText(date_time, in);
if (!in.eof())
throw Exception("String is too long for DateTime: " + column_string->getData());
ColumnConst<DataTypeDateTime::FieldType> parsed_const_date_time(block.rowsInFirstColumn(), date_time);
executeNumLeftType<DataTypeDateTime::FieldType>(block, result,
left_is_num ? col_left_untyped : &parsed_const_date_time,
left_is_num ? &parsed_const_date_time : col_right_untyped);
}
}
public: public:
/// Получить имя функции. /// Получить имя функции.
String getName() const String getName() const
@ -581,12 +635,36 @@ public:
+ toString(arguments.size()) + ", should be 2.", + toString(arguments.size()) + ", should be 2.",
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH); ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
if (!( ( arguments[0]->isNumeric() && arguments[0]->behavesAsNumber() bool left_is_date = false;
&& arguments[1]->isNumeric() && arguments[1]->behavesAsNumber()) bool left_is_date_time = false;
|| ( (arguments[0]->getName() == "String" || arguments[0]->getName().substr(0, 11) == "FixedString") bool left_is_string = false;
&& (arguments[1]->getName() == "String" || arguments[1]->getName().substr(0, 11) == "FixedString")) bool left_is_fixed_string = false;
|| (arguments[0]->getName() == "Date" && arguments[1]->getName() == "Date")
|| (arguments[0]->getName() == "DateTime" && arguments[1]->getName() == "DateTime"))) false
|| (left_is_date = typeid_cast<const DataTypeDate *>(arguments[0].get()))
|| (left_is_date_time = typeid_cast<const DataTypeDateTime *>(arguments[0].get()))
|| (left_is_string = typeid_cast<const DataTypeString *>(arguments[0].get()))
|| (left_is_fixed_string = typeid_cast<const DataTypeFixedString *>(arguments[0].get()));
bool right_is_date = false;
bool right_is_date_time = false;
bool right_is_string = false;
bool right_is_fixed_string = false;
false
|| (right_is_date = typeid_cast<const DataTypeDate *>(arguments[1].get()))
|| (right_is_date_time = typeid_cast<const DataTypeDateTime *>(arguments[1].get()))
|| (right_is_string = typeid_cast<const DataTypeString *>(arguments[1].get()))
|| (right_is_fixed_string = typeid_cast<const DataTypeFixedString *>(arguments[1].get()));
if (!( (arguments[0]->behavesAsNumber() && arguments[1]->behavesAsNumber())
|| ((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) /// Можно сравнивать дату и дату-с-временем с константной строкой.
|| (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)))
throw Exception("Illegal types of arguments (" + arguments[0]->getName() + ", " + arguments[1]->getName() + ")" throw Exception("Illegal types of arguments (" + arguments[0]->getName() + ", " + arguments[1]->getName() + ")"
" of function " + getName(), ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT); " of function " + getName(), ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
@ -596,24 +674,36 @@ public:
/// Выполнить функцию над блоком. /// Выполнить функцию над блоком.
void execute(Block & block, const ColumnNumbers & arguments, size_t result) void execute(Block & block, const ColumnNumbers & arguments, size_t result)
{ {
if (block.getByPosition(arguments[0]).column->isNumeric()) const IColumn * col_left_untyped = block.getByPosition(arguments[0]).column.get();
const IColumn * col_right_untyped = block.getByPosition(arguments[1]).column.get();
bool left_is_num = col_left_untyped->isNumeric();
bool right_is_num = col_right_untyped->isNumeric();
if (left_is_num && right_is_num)
{ {
if (!( executeNumLeftType<UInt8>(block, arguments, result) if (!( executeNumLeftType<UInt8>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<UInt16>(block, arguments, result) || executeNumLeftType<UInt16>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<UInt32>(block, arguments, result) || executeNumLeftType<UInt32>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<UInt64>(block, arguments, result) || executeNumLeftType<UInt64>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<Int8>(block, arguments, result) || executeNumLeftType<Int8>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<Int16>(block, arguments, result) || executeNumLeftType<Int16>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<Int32>(block, arguments, result) || executeNumLeftType<Int32>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<Int64>(block, arguments, result) || executeNumLeftType<Int64>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<Float32>(block, arguments, result) || executeNumLeftType<Float32>(block, result, col_left_untyped, col_right_untyped)
|| executeNumLeftType<Float64>(block, arguments, result))) || executeNumLeftType<Float64>(block, result, col_left_untyped, col_right_untyped)))
throw Exception("Illegal column " + block.getByPosition(arguments[0]).column->getName() throw Exception("Illegal column " + col_left_untyped->getName()
+ " of first argument of function " + getName(), + " of first argument of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN); ErrorCodes::ILLEGAL_COLUMN);
} }
else if (!left_is_num && !right_is_num)
{
executeString(block, result, col_left_untyped, col_right_untyped);
}
else else
executeString(block, arguments, result); {
executeDateOrDateTimeWithConstString(block, result, col_left_untyped, col_right_untyped, left_is_num, right_is_num);
}
} }
}; };

View File

@ -12,6 +12,7 @@
#include <DB/DataTypes/DataTypeDateTime.h> #include <DB/DataTypes/DataTypeDateTime.h>
#include <DB/DataTypes/DataTypeTuple.h> #include <DB/DataTypes/DataTypeTuple.h>
#include <DB/DataTypes/DataTypeArray.h> #include <DB/DataTypes/DataTypeArray.h>
#include <DB/DataTypes/DataTypeAggregateFunction.h>
#include <DB/Columns/ColumnString.h> #include <DB/Columns/ColumnString.h>
#include <DB/Columns/ColumnFixedString.h> #include <DB/Columns/ColumnFixedString.h>
#include <DB/Columns/ColumnConst.h> #include <DB/Columns/ColumnConst.h>
@ -20,6 +21,7 @@
#include <DB/Columns/ColumnTuple.h> #include <DB/Columns/ColumnTuple.h>
#include <DB/Columns/ColumnArray.h> #include <DB/Columns/ColumnArray.h>
#include <DB/Columns/ColumnReplicated.h> #include <DB/Columns/ColumnReplicated.h>
#include <DB/Columns/ColumnAggregateFunction.h>
#include <DB/Common/UnicodeBar.h> #include <DB/Common/UnicodeBar.h>
#include <DB/Functions/IFunction.h> #include <DB/Functions/IFunction.h>
#include <DB/Interpreters/ExpressionActions.h> #include <DB/Interpreters/ExpressionActions.h>
@ -58,6 +60,11 @@ namespace DB
* bar(x, min, max, width) - рисует полосу из количества символов, пропорционального (x - min) и равного width при x == max. * bar(x, min, max, width) - рисует полосу из количества символов, пропорционального (x - min) и равного width при x == max.
* *
* version() - возвращает текущую версию сервера в строке. * version() - возвращает текущую версию сервера в строке.
*
* finalizeAggregation(agg_state) - по состоянию агрегации получить результат.
*
* runningAccumulate(agg_state) - принимает состояния агрегатной функции и возвращает столбец со значениями,
* являющимися результатом накопления этих состояний для множества строк блока, от первой до текущей строки.
*/ */
@ -890,6 +897,7 @@ using FunctionIsFinite = FunctionNumericPredicate<IsFiniteImpl>;
using FunctionIsInfinite = FunctionNumericPredicate<IsInfiniteImpl>; using FunctionIsInfinite = FunctionNumericPredicate<IsInfiniteImpl>;
using FunctionIsNaN = FunctionNumericPredicate<IsNaNImpl>; using FunctionIsNaN = FunctionNumericPredicate<IsNaNImpl>;
class FunctionVersion : public IFunction class FunctionVersion : public IFunction
{ {
public: public:
@ -920,4 +928,98 @@ private:
} }
}; };
/** Весьма необычная функция.
* Принимает состояние агрегатной функции (например runningAccumulate(uniqState(UserID))),
* и для каждой строки блока, возвращает результат агрегатной функции по объединению состояний от всех предыдущих строк блока и текущей строки.
*
* То есть, функция зависит от разбиения данных на блоки и от порядка строк в блоке.
*/
class FunctionRunningAccumulate : public IFunction
{
public:
static constexpr auto name = "runningAccumulate";
static IFunction * create(const Context & context) { return new FunctionRunningAccumulate; }
String getName() const override { return name; }
DataTypePtr getReturnType(const DataTypes & arguments) const override
{
if (arguments.size() != 1)
throw Exception("Function " + getName() + " requires exactly one argument.", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
const DataTypeAggregateFunction * type = typeid_cast<const DataTypeAggregateFunction *>(&*arguments[0]);
if (!type)
throw Exception("Argument for function " + getName() + " must have type AggregateFunction - state of aggregate function.",
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
return type->getReturnType()->clone();
}
void execute(Block & block, const ColumnNumbers & arguments, size_t result) override
{
const ColumnAggregateFunction * column_with_states = typeid_cast<const ColumnAggregateFunction *>(&*block.getByPosition(arguments.at(0)).column);
if (!column_with_states)
throw Exception(
"Illegal column " + block.getByPosition(arguments.at(0)).column->getName() + " of first argument of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN);
AggregateFunctionPtr aggregate_function_ptr = column_with_states->getAggregateFunction();
const IAggregateFunction & agg_func = *aggregate_function_ptr;
auto deleter = [&agg_func] (char * ptr) { agg_func.destroy(ptr); free(ptr); };
std::unique_ptr<char, decltype(deleter)> place { reinterpret_cast<char *>(malloc(agg_func.sizeOfData())), deleter };
agg_func.create(place.get()); /// Немного не exception-safe. Если здесь выкинется исключение, то зря вызовется destroy.
ColumnPtr result_column_ptr = agg_func.getReturnType()->createColumn();
block.getByPosition(result).column = result_column_ptr;
IColumn & result_column = *result_column_ptr;
result_column.reserve(column_with_states->size());
const auto & states = column_with_states->getData();
for (const auto & state_to_add : states)
{
agg_func.merge(place.get(), state_to_add);
agg_func.insertResultInto(place.get(), result_column);
}
}
};
/** Принимает состояние агрегатной функции. Возвращает результат агрегации.
*/
class FunctionFinalizeAggregation : public IFunction
{
public:
static constexpr auto name = "finalizeAggregation";
static IFunction * create(const Context & context) { return new FunctionFinalizeAggregation; }
String getName() const override { return name; }
DataTypePtr getReturnType(const DataTypes & arguments) const override
{
if (arguments.size() != 1)
throw Exception("Function " + getName() + " requires exactly one argument.", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
const DataTypeAggregateFunction * type = typeid_cast<const DataTypeAggregateFunction *>(&*arguments[0]);
if (!type)
throw Exception("Argument for function " + getName() + " must have type AggregateFunction - state of aggregate function.",
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
return type->getReturnType()->clone();
}
void execute(Block & block, const ColumnNumbers & arguments, size_t result) override
{
ColumnAggregateFunction * column_with_states = typeid_cast<ColumnAggregateFunction *>(&*block.getByPosition(arguments.at(0)).column);
if (!column_with_states)
throw Exception(
"Illegal column " + block.getByPosition(arguments.at(0)).column->getName() + " of first argument of function " + getName(),
ErrorCodes::ILLEGAL_COLUMN);
block.getByPosition(result).column = column_with_states->convertToValues();
}
};
} }

View File

@ -202,46 +202,11 @@ struct LengthUTF8Impl
}; };
/** Переводит строку в нижний (верхний) регистр, в текущей локали, в однобайтовой кодировке. template <char not_case_lower_bound, char not_case_upper_bound>
*/
template <int F(int)>
struct LowerUpperImpl struct LowerUpperImpl
{ {
static void vector(const ColumnString::Chars_t & data, const ColumnString::Offsets_t & offsets, static void vector(const ColumnString::Chars_t & data, const ColumnString::Offsets_t & offsets,
ColumnString::Chars_t & res_data, ColumnString::Offsets_t & res_offsets) ColumnString::Chars_t & res_data, ColumnString::Offsets_t & res_offsets)
{
res_data.resize(data.size());
res_offsets.assign(offsets);
array(&*data.begin(), &*data.end(), &*res_data.begin());
}
static void vector_fixed(const ColumnString::Chars_t & data, size_t n,
ColumnString::Chars_t & res_data)
{
res_data.resize(data.size());
array(&*data.begin(), &*data.end(), &*res_data.begin());
}
static void constant(const std::string & data, std::string & res_data)
{
res_data.resize(data.size());
array(reinterpret_cast<const UInt8 *>(&*data.begin()), reinterpret_cast<const UInt8 *>(&*data.end()),
reinterpret_cast<UInt8 *>(&*res_data.begin()));
}
private:
static void array(const UInt8 * src, const UInt8 * src_end, UInt8 * dst)
{
for (; src < src_end; ++src, ++dst)
*dst = F(*src);
}
};
template <char not_case_lower_bound, char not_case_upper_bound>
struct LowerUpperImplVectorized
{
static void vector(const ColumnString::Chars_t & data, const ColumnString::Offsets_t & offsets,
ColumnString::Chars_t & res_data, ColumnString::Offsets_t & res_offsets)
{ {
res_data.resize(data.size()); res_data.resize(data.size());
res_offsets.assign(offsets); res_offsets.assign(offsets);
@ -348,9 +313,14 @@ inline void UTF8CyrillicToCase(const UInt8 * & src, const UInt8 * const src_end,
} }
}; };
/** Если строка содержит текст в кодировке UTF-8 - перевести его в нижний (верхний) регистр.
* Замечание: предполагается, что после перевода символа в другой регистр,
* длина его мультибайтовой последовательности в UTF-8 не меняется.
* Иначе - поведение не определено.
*/
template <char not_case_lower_bound, char not_case_upper_bound, template <char not_case_lower_bound, char not_case_upper_bound,
int to_case(int), void cyrillic_to_case(const UInt8 * &, const UInt8 *, UInt8 * &)> int to_case(int), void cyrillic_to_case(const UInt8 * &, const UInt8 *, UInt8 * &)>
struct LowerUpperUTF8ImplVectorized struct LowerUpperUTF8Impl
{ {
static void vector(const ColumnString::Chars_t & data, const ColumnString::Offsets_t & offsets, static void vector(const ColumnString::Chars_t & data, const ColumnString::Offsets_t & offsets,
ColumnString::Chars_t & res_data, ColumnString::Offsets_t & res_offsets) ColumnString::Chars_t & res_data, ColumnString::Offsets_t & res_offsets)
@ -487,59 +457,6 @@ private:
}; };
/** Если строка содержит текст в кодировке UTF-8 - перевести его в нижний (верхний) регистр.
* Замечание: предполагается, что после перевода символа в другой регистр,
* длина его мультибайтовой последовательности в UTF-8 не меняется.
* Иначе - поведение не определено.
*/
template <int F(int)>
struct LowerUpperUTF8Impl
{
static void vector(const ColumnString::Chars_t & data, const ColumnString::Offsets_t & offsets,
ColumnString::Chars_t & res_data, ColumnString::Offsets_t & res_offsets)
{
res_data.resize(data.size());
res_offsets.assign(offsets);
array(&*data.begin(), &*data.end(), &*res_data.begin());
}
static void vector_fixed(const ColumnString::Chars_t & data, size_t n,
ColumnString::Chars_t & res_data)
{
res_data.resize(data.size());
array(&*data.begin(), &*data.end(), &*res_data.begin());
}
static void constant(const std::string & data, std::string & res_data)
{
res_data.resize(data.size());
array(reinterpret_cast<const UInt8 *>(&*data.begin()), reinterpret_cast<const UInt8 *>(&*data.end()),
reinterpret_cast<UInt8 *>(&*res_data.begin()));
}
private:
static void array(const UInt8 * src, const UInt8 * src_end, UInt8 * dst)
{
static Poco::UTF8Encoding utf8;
while (src < src_end)
{
int chars = utf8.convert(F(utf8.convert(src)), dst, src_end - src);
if (chars)
{
src += chars;
dst += chars;
}
else
{
++src;
++dst;
}
}
}
};
/** Разворачивает строку в байтах. /** Разворачивает строку в байтах.
*/ */
struct ReverseImpl struct ReverseImpl
@ -1676,32 +1593,22 @@ struct NameReverseUTF8 { static constexpr auto name = "reverseUTF8"; };
struct NameSubstring { static constexpr auto name = "substring"; }; struct NameSubstring { static constexpr auto name = "substring"; };
struct NameSubstringUTF8 { static constexpr auto name = "substringUTF8"; }; struct NameSubstringUTF8 { static constexpr auto name = "substringUTF8"; };
struct NameSSELower { static constexpr auto name = "sse_lower"; };
struct NameSSEUpper { static constexpr auto name = "sse_upper"; };
struct NameSSELowerUTF8 { static constexpr auto name = "sse_lowerUTF8"; };
struct NameSSEUpperUTF8 { static constexpr auto name = "sse_upperUTF8"; };
typedef FunctionStringOrArrayToT<EmptyImpl<false>, NameEmpty, UInt8> FunctionEmpty; typedef FunctionStringOrArrayToT<EmptyImpl<false>, NameEmpty, UInt8> FunctionEmpty;
typedef FunctionStringOrArrayToT<EmptyImpl<true>, NameNotEmpty, UInt8> FunctionNotEmpty; typedef FunctionStringOrArrayToT<EmptyImpl<true>, NameNotEmpty, UInt8> FunctionNotEmpty;
typedef FunctionStringOrArrayToT<LengthImpl, NameLength, UInt64> FunctionLength; typedef FunctionStringOrArrayToT<LengthImpl, NameLength, UInt64> FunctionLength;
typedef FunctionStringOrArrayToT<LengthUTF8Impl, NameLengthUTF8, UInt64> FunctionLengthUTF8; typedef FunctionStringOrArrayToT<LengthUTF8Impl, NameLengthUTF8, UInt64> FunctionLengthUTF8;
typedef FunctionStringToString<LowerUpperImpl<tolower>, NameLower> FunctionLower; typedef FunctionStringToString<LowerUpperImpl<'A', 'Z'>, NameLower> FunctionLower;
typedef FunctionStringToString<LowerUpperImpl<toupper>, NameUpper> FunctionUpper; typedef FunctionStringToString<LowerUpperImpl<'a', 'z'>, NameUpper> FunctionUpper;
typedef FunctionStringToString<LowerUpperUTF8Impl<Poco::Unicode::toLower>, NameLowerUTF8> FunctionLowerUTF8; typedef FunctionStringToString<
typedef FunctionStringToString<LowerUpperUTF8Impl<Poco::Unicode::toUpper>, NameUpperUTF8> FunctionUpperUTF8; LowerUpperUTF8Impl<'A', 'Z', Poco::Unicode::toLower, UTF8CyrillicToCase<true>>,
NameLowerUTF8> FunctionLowerUTF8;
typedef FunctionStringToString<
LowerUpperUTF8Impl<'a', 'z', Poco::Unicode::toUpper, UTF8CyrillicToCase<false>>,
NameUpperUTF8> FunctionUpperUTF8;
typedef FunctionStringToString<ReverseImpl, NameReverse> FunctionReverse; typedef FunctionStringToString<ReverseImpl, NameReverse> FunctionReverse;
typedef FunctionStringToString<ReverseUTF8Impl, NameReverseUTF8> FunctionReverseUTF8; typedef FunctionStringToString<ReverseUTF8Impl, NameReverseUTF8> FunctionReverseUTF8;
typedef FunctionStringNumNumToString<SubstringImpl, NameSubstring> FunctionSubstring; typedef FunctionStringNumNumToString<SubstringImpl, NameSubstring> FunctionSubstring;
typedef FunctionStringNumNumToString<SubstringUTF8Impl, NameSubstringUTF8> FunctionSubstringUTF8; typedef FunctionStringNumNumToString<SubstringUTF8Impl, NameSubstringUTF8> FunctionSubstringUTF8;
using FunctionSSELower = FunctionStringToString<LowerUpperImplVectorized<'A', 'Z'>, NameSSELower>;
using FunctionSSEUpper = FunctionStringToString<LowerUpperImplVectorized<'a', 'z'>, NameSSEUpper>;
using FunctionSSELowerUTF8 = FunctionStringToString<
LowerUpperUTF8ImplVectorized<'A', 'Z', Poco::Unicode::toLower, UTF8CyrillicToCase<true>>,
NameSSELowerUTF8>;
using FunctionSSEUpperUTF8 = FunctionStringToString<
LowerUpperUTF8ImplVectorized<'a', 'z', Poco::Unicode::toUpper, UTF8CyrillicToCase<false>>,
NameSSEUpperUTF8>;
} }

View File

@ -1,23 +1,23 @@
#pragma once #pragma once
#include <DB/Dictionaries/IDictionary.h>
#include <DB/Core/Exception.h> #include <DB/Core/Exception.h>
#include <DB/Core/ErrorCodes.h> #include <DB/Core/ErrorCodes.h>
#include <Yandex/MultiVersion.h> #include <Yandex/MultiVersion.h>
#include <Yandex/logger_useful.h> #include <Yandex/logger_useful.h>
#include <Poco/Event.h> #include <Poco/Event.h>
#include <unistd.h>
#include <time.h> #include <time.h>
#include <mutex> #include <mutex>
#include <thread> #include <thread>
#include <unordered_map> #include <unordered_map>
#include <chrono> #include <chrono>
#include <random> #include <random>
#include <unistd.h>
namespace DB namespace DB
{ {
class Context; class Context;
class IDictionary;
/** Manages user-defined dictionaries. /** Manages user-defined dictionaries.
* Monitors configuration file and automatically reloads dictionaries in a separate thread. * Monitors configuration file and automatically reloads dictionaries in a separate thread.
@ -50,8 +50,16 @@ private:
std::exception_ptr exception; std::exception_ptr exception;
}; };
struct failed_dictionary_info final
{
std::unique_ptr<IDictionary> dict;
std::chrono::system_clock::time_point next_attempt_time;
std::uint64_t error_count;
};
std::unordered_map<std::string, dictionary_info> dictionaries; std::unordered_map<std::string, dictionary_info> dictionaries;
std::unordered_map<std::string, std::chrono::system_clock::time_point> update_times; std::unordered_map<std::string, std::chrono::system_clock::time_point> update_times;
std::unordered_map<std::string, failed_dictionary_info> failed_dictionaries;
std::mt19937_64 rnd_engine{getSeed()}; std::mt19937_64 rnd_engine{getSeed()};
Context & context; Context & context;
@ -81,7 +89,7 @@ private:
{ {
timespec ts; timespec ts;
clock_gettime(CLOCK_MONOTONIC, &ts); clock_gettime(CLOCK_MONOTONIC, &ts);
return ts.tv_nsec ^ getpid(); return static_cast<std::uint64_t>(ts.tv_nsec ^ getpid());
} }
public: public:

View File

@ -130,6 +130,15 @@ private:
void ignoreWithTotals(); void ignoreWithTotals();
/** Если в запросе SELECT есть секция SETTINGS, то применить настройки из неё.
* Затем достать настройки из context и поместить их в settings.
*
* Секция SETTINGS - настройки для конкретного запроса.
* Обычно настройки могут быть переданы другими способами, не внутри запроса.
* Но использование такой секции оправдано, если нужно задать настройки для одного подзапроса.
*/
void initSettings();
ASTPtr query_ptr; ASTPtr query_ptr;
ASTSelectQuery & query; ASTSelectQuery & query;
Context context; Context context;

View File

@ -8,7 +8,7 @@ namespace DB
{ {
/** Установить один или несколько параметров, для сессии или глобально. /** Установить один или несколько параметров, для сессии или глобально... или для текущего запроса.
*/ */
class InterpreterSetQuery class InterpreterSetQuery
{ {
@ -16,17 +16,36 @@ public:
InterpreterSetQuery(ASTPtr query_ptr_, Context & context_) InterpreterSetQuery(ASTPtr query_ptr_, Context & context_)
: query_ptr(query_ptr_), context(context_) {} : query_ptr(query_ptr_), context(context_) {}
/** Обычный запрос SET. Задать настройку на сессию или глобальную (если указано GLOBAL).
*/
void execute() void execute()
{ {
ASTSetQuery & ast = typeid_cast<ASTSetQuery &>(*query_ptr); ASTSetQuery & ast = typeid_cast<ASTSetQuery &>(*query_ptr);
Context & target = ast.global ? context.getGlobalContext() : context.getSessionContext(); Context & target = ast.global ? context.getGlobalContext() : context.getSessionContext();
executeImpl(ast, target);
}
/** Задать настроку для текущего контекста (контекста запроса).
* Используется для интерпретации секции SETTINGS в запросе SELECT.
*/
void executeForCurrentContext()
{
ASTSetQuery & ast = typeid_cast<ASTSetQuery &>(*query_ptr);
executeImpl(ast, context);
}
private:
ASTPtr query_ptr;
Context & context;
void executeImpl(ASTSetQuery & ast, Context & target)
{
/** Значение readonly понимается следующим образом: /** Значение readonly понимается следующим образом:
* 0 - можно всё. * 0 - можно всё.
* 1 - можно делать только запросы на чтение; в том числе, нельзя менять настройки. * 1 - можно делать только запросы на чтение; в том числе, нельзя менять настройки.
* 2 - можно делать только запросы на чтение и можно менять настройки, кроме настройки readonly. * 2 - можно делать только запросы на чтение и можно менять настройки, кроме настройки readonly.
*/ */
if (context.getSettingsRef().limits.readonly == 1) if (context.getSettingsRef().limits.readonly == 1)
throw Exception("Cannot execute SET query in readonly mode", ErrorCodes::READONLY); throw Exception("Cannot execute SET query in readonly mode", ErrorCodes::READONLY);
@ -39,10 +58,6 @@ public:
for (ASTSetQuery::Changes::const_iterator it = ast.changes.begin(); it != ast.changes.end(); ++it) for (ASTSetQuery::Changes::const_iterator it = ast.changes.begin(); it != ast.changes.end(); ++it)
target.setSetting(it->name, it->value); target.setSetting(it->name, it->value);
} }
private:
ASTPtr query_ptr;
Context & context;
}; };

View File

@ -222,7 +222,8 @@ private:
bool keys_fit_128_bits; bool keys_fit_128_bits;
Sizes key_sizes; Sizes key_sizes;
Block sample_block; Block sample_block_with_columns_to_add;
Block sample_block_with_keys;
Logger * log; Logger * log;
@ -250,6 +251,9 @@ private:
/// Проверить не превышены ли допустимые размеры множества /// Проверить не превышены ли допустимые размеры множества
bool checkSizeLimits() const; bool checkSizeLimits() const;
/// Кинуть исключение, если в блоках не совпадают типы ключей.
void checkTypesOfKeys(const Block & block_left, const Block & block_right) const;
}; };
typedef Poco::SharedPtr<Join> JoinPtr; typedef Poco::SharedPtr<Join> JoinPtr;

View File

@ -283,7 +283,7 @@ public:
* node - это список значений: 1, 2, 3 или список tuple-ов: (1, 2), (3, 4), (5, 6). * node - это список значений: 1, 2, 3 или список tuple-ов: (1, 2), (3, 4), (5, 6).
* create_ordered_set - создавать ли вектор упорядоченных элементов. Нужен для работы индекса * create_ordered_set - создавать ли вектор упорядоченных элементов. Нужен для работы индекса
*/ */
void createFromAST(DataTypes & types, ASTPtr node, bool create_ordered_set); void createFromAST(DataTypes & types, ASTPtr node, const Context & context, bool create_ordered_set);
// Возвращает false, если превышено какое-нибудь ограничение, и больше не нужно вставлять. // Возвращает false, если превышено какое-нибудь ограничение, и больше не нужно вставлять.
bool insertFromBlock(const Block & block, bool create_ordered_set = false); bool insertFromBlock(const Block & block, bool create_ordered_set = false);

View File

@ -51,6 +51,7 @@ public:
ASTPtr order_expression_list; ASTPtr order_expression_list;
ASTPtr limit_offset; ASTPtr limit_offset;
ASTPtr limit_length; ASTPtr limit_length;
ASTPtr settings;
ASTPtr next_union_all; /// Следующий запрос SELECT в цепочке UNION ALL, если такой есть ASTPtr next_union_all; /// Следующий запрос SELECT в цепочке UNION ALL, если такой есть
}; };

View File

@ -12,9 +12,15 @@ namespace DB
*/ */
class ParserSetQuery : public IParserBase class ParserSetQuery : public IParserBase
{ {
public:
ParserSetQuery(bool parse_only_internals_ = false) : parse_only_internals(parse_only_internals_) {}
protected: protected:
const char * getName() const { return "SET query"; } const char * getName() const { return "SET query"; }
bool parseImpl(Pos & pos, Pos end, ASTPtr & node, Pos & max_parsed_pos, Expected & expected); bool parseImpl(Pos & pos, Pos end, ASTPtr & node, Pos & max_parsed_pos, Expected & expected);
/// Парсить список name = value пар, без SET [GLOBAL].
bool parse_only_internals;
}; };
} }

View File

@ -6,6 +6,7 @@
#include <DB/Core/Exception.h> #include <DB/Core/Exception.h>
#include <DB/Core/ErrorCodes.h> #include <DB/Core/ErrorCodes.h>
#include <DB/IO/WriteHelpers.h> #include <DB/IO/WriteHelpers.h>
#include <DB/Common/formatReadable.h>
namespace DB namespace DB
{ {
@ -116,8 +117,8 @@ public:
{ {
size_t free_bytes = getUnreservedFreeSpace(path); size_t free_bytes = getUnreservedFreeSpace(path);
if (free_bytes < size) if (free_bytes < size)
throw Exception("Not enough free disk space to reserve: " + toString(free_bytes) + " available, " throw Exception("Not enough free disk space to reserve: " + formatReadableSizeWithBinarySuffix(free_bytes) + " available, "
+ toString(size) + " requested", ErrorCodes::NOT_ENOUGH_SPACE); + formatReadableSizeWithBinarySuffix(size) + " requested", ErrorCodes::NOT_ENOUGH_SPACE);
return new Reservation(size); return new Reservation(size);
} }

View File

@ -43,9 +43,7 @@ public:
{ {
pre_column_names = prewhere_actions->getRequiredColumns(); pre_column_names = prewhere_actions->getRequiredColumns();
/// @todo somehow decide which injected columns belong to PREWHERE, optimizing reads injectRequiredColumns(pre_column_names);
pre_column_names.insert(std::end(pre_column_names),
std::begin(injected_columns), std::end(injected_columns));
if (pre_column_names.empty()) if (pre_column_names.empty())
pre_column_names.push_back(column_names[0]); pre_column_names.push_back(column_names[0]);
@ -100,7 +98,7 @@ public:
setTotalRowsApprox(total_rows); setTotalRowsApprox(total_rows);
} }
String getName() const override { return "MergeTreeBlockInputStream"; } String getName() const override { return "MergeTree"; }
String getID() const override String getID() const override
{ {

View File

@ -15,7 +15,17 @@ public:
void writePrefix() override void writePrefix() override
{ {
storage.data.delayInsertIfNeeded(); /// Если слишком много кусков - делаем внеочередные мерджи, синхронно, в текущем потоке.
/// Почему 10? - на всякий случай, вместо бесконечного цикла.
for (size_t i = 0; i < 10; ++i)
{
size_t parts_count = storage.data.getMaxPartsCountForMonth();
if (parts_count <= storage.data.settings.parts_to_delay_insert)
break;
ProfileEvents::increment(ProfileEvents::SynchronousMergeOnInsert);
storage.merge(0, true);
}
} }
void write(const Block & block) override void write(const Block & block) override
@ -26,6 +36,8 @@ public:
UInt64 temp_index = storage.increment.get(); UInt64 temp_index = storage.increment.get();
MergeTreeData::MutableDataPartPtr part = storage.writer.writeTempPart(current_block, temp_index); MergeTreeData::MutableDataPartPtr part = storage.writer.writeTempPart(current_block, temp_index);
storage.data.renameTempPartAndAdd(part, &storage.increment); storage.data.renameTempPartAndAdd(part, &storage.increment);
/// Инициируем асинхронный мердж - он будет произведён, если пора делать мердж и если в background_pool-е есть место.
storage.merge_task_handle->wake(); storage.merge_task_handle->wake();
} }
} }

View File

@ -49,7 +49,7 @@ public:
DiskSpaceMonitor::Reservation * disk_reservation = nullptr); DiskSpaceMonitor::Reservation * disk_reservation = nullptr);
/// Примерное количество места на диске, нужное для мерджа. С запасом. /// Примерное количество места на диске, нужное для мерджа. С запасом.
size_t estimateDiskSpaceForMerge(const MergeTreeData::DataPartsVector & parts); static size_t estimateDiskSpaceForMerge(const MergeTreeData::DataPartsVector & parts);
/** Отменяет все мерджи. Все выполняющиеся сейчас вызовы mergeParts скоро бросят исключение. /** Отменяет все мерджи. Все выполняющиеся сейчас вызовы mergeParts скоро бросят исключение.
* Все новые вызовы будут бросать исключения, пока не будет вызван uncancelAll(). * Все новые вызовы будут бросать исключения, пока не будет вызван uncancelAll().

View File

@ -9,12 +9,14 @@
#include <DB/Parsers/ASTSubquery.h> #include <DB/Parsers/ASTSubquery.h>
#include <DB/Parsers/formatAST.h> #include <DB/Parsers/formatAST.h>
#include <DB/Common/escapeForFileName.h> #include <DB/Common/escapeForFileName.h>
#include <statdaemons/ext/scope_guard.hpp>
#include <memory> #include <memory>
#include <unordered_map> #include <unordered_map>
#include <map> #include <map>
#include <limits> #include <limits>
#include <cstddef> #include <cstddef>
namespace DB namespace DB
{ {
@ -34,6 +36,7 @@ class MergeTreeWhereOptimizer
static constexpr auto max_columns_relative_size = 0.25f; static constexpr auto max_columns_relative_size = 0.25f;
static constexpr auto and_function_name = "and"; static constexpr auto and_function_name = "and";
static constexpr auto equals_function_name = "equals"; static constexpr auto equals_function_name = "equals";
static constexpr auto array_join_function_name = "arrayJoin";
public: public:
MergeTreeWhereOptimizer(const MergeTreeWhereOptimizer&) = delete; MergeTreeWhereOptimizer(const MergeTreeWhereOptimizer&) = delete;
@ -46,6 +49,7 @@ public:
table_columns{toUnorderedSet(data.getColumnsList())}, log{log} table_columns{toUnorderedSet(data.getColumnsList())}, log{log}
{ {
calculateColumnSizes(data, column_names); calculateColumnSizes(data, column_names);
determineArrayJoinedNames(select);
optimize(select); optimize(select);
} }
@ -96,28 +100,33 @@ private:
{ {
const auto condition = conditions[idx].get(); const auto condition = conditions[idx].get();
/// linearize sub-conjunctions
if (const auto function = typeid_cast<ASTFunction *>(condition))
{
if (function->name == and_function_name)
{
for (auto & child : function->arguments->children)
conditions.emplace_back(std::move(child));
/// remove the condition corresponding to conjunction
remove_condition_at_index(idx);
/// continue iterating without increment to ensure the just added conditions are processed
continue;
}
}
SCOPE_EXIT(++idx);
if (cannotBeMoved(condition))
continue;
IdentifierNameSet identifiers{}; IdentifierNameSet identifiers{};
collectIdentifiersNoSubqueries(condition, identifiers); collectIdentifiersNoSubqueries(condition, identifiers);
/// do not take into consideration the conditions consisting only of primary key columns /// do not take into consideration the conditions consisting only of primary key columns
if (hasNonPrimaryKeyColumns(identifiers) && isSubsetOfTableColumns(identifiers)) if (hasNonPrimaryKeyColumns(identifiers) && isSubsetOfTableColumns(identifiers))
{ {
/// linearize sub-conjunctions
if (const auto function = typeid_cast<ASTFunction *>(condition))
{
if (function->name == and_function_name)
{
for (auto & child : function->arguments->children)
conditions.emplace_back(std::move(child));
/// remove the condition corresponding to conjunction
remove_condition_at_index(idx);
/// continue iterating without increment to ensure the just added conditions are processed
continue;
}
}
/// calculate size of columns involved in condition /// calculate size of columns involved in condition
const auto cond_columns_size = getIdentifiersColumnSize(identifiers); const auto cond_columns_size = getIdentifiersColumnSize(identifiers);
@ -129,8 +138,6 @@ private:
good_or_viable_condition.second = cond_columns_size; good_or_viable_condition.second = cond_columns_size;
} }
} }
++idx;
} }
const auto move_condition_to_prewhere = [&] (const std::size_t idx) { const auto move_condition_to_prewhere = [&] (const std::size_t idx) {
@ -180,6 +187,10 @@ private:
{ {
auto & condition = select.where_expression; auto & condition = select.where_expression;
/// do not optimize restricted expressions
if (cannotBeMoved(select.where_expression.get()))
return;
IdentifierNameSet identifiers{}; IdentifierNameSet identifiers{};
collectIdentifiersNoSubqueries(condition, identifiers); collectIdentifiersNoSubqueries(condition, identifiers);
@ -300,11 +311,49 @@ private:
return true; return true;
} }
/** ARRAY JOIN'ed columns as well as arrayJoin() result cannot be used in PREWHERE, therefore expressions
* containing said columns should not be moved to PREWHERE at all.
* We assume all AS aliases have been expanded prior to using this class */
bool cannotBeMoved(const IAST * ptr) const
{
if (const auto function_ptr = typeid_cast<const ASTFunction *>(ptr))
{
/// disallow arrayJoin expressions to be moved to PREWHERE for now
if (array_join_function_name == function_ptr->name)
return true;
}
else if (const auto identifier_ptr = typeid_cast<const ASTIdentifier *>(ptr))
{
/// disallow moving result of ARRAY JOIN to PREWHERE
if (identifier_ptr->kind == ASTIdentifier::Column)
if (array_joined_names.count(identifier_ptr->name) ||
array_joined_names.count(DataTypeNested::extractNestedTableName(identifier_ptr->name)))
return true;
}
for (const auto & child : ptr->children)
if (cannotBeMoved(child.get()))
return true;
return false;
}
void determineArrayJoinedNames(ASTSelectQuery & select)
{
/// much simplified code from ExpressionAnalyzer::getArrayJoinedColumns()
if (!select.array_join_expression_list)
return;
for (const auto & ast : select.array_join_expression_list->children)
array_joined_names.emplace(ast->getAliasOrColumnName());
}
string_set_t primary_key_columns{}; string_set_t primary_key_columns{};
string_set_t table_columns{}; string_set_t table_columns{};
Logger * log; Logger * log;
std::unordered_map<std::string, std::size_t> column_sizes{}; std::unordered_map<std::string, std::size_t> column_sizes{};
std::size_t total_column_size{}; std::size_t total_column_size{};
NameSet array_joined_names;
}; };

View File

@ -48,7 +48,11 @@ public:
/** num_shards - уровень внутреннего параллелизма (количество независимых буферов) /** num_shards - уровень внутреннего параллелизма (количество независимых буферов)
* Буфер сбрасывается, если превышены все минимальные пороги или хотя бы один из максимальных. * Буфер сбрасывается, если превышены все минимальные пороги или хотя бы один из максимальных.
*/ */
static StoragePtr create(const std::string & name_, NamesAndTypesListPtr columns_, Context & context_, static StoragePtr create(const std::string & name_, NamesAndTypesListPtr columns_,
const NamesAndTypesList & materialized_columns_,
const NamesAndTypesList & alias_columns_,
const ColumnDefaults & column_defaults_,
Context & context_,
size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_, size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_,
const String & destination_database_, const String & destination_table_); const String & destination_database_, const String & destination_table_);
@ -113,7 +117,11 @@ private:
/// Выполняет сброс данных по таймауту. /// Выполняет сброс данных по таймауту.
std::thread flush_thread; std::thread flush_thread;
StorageBuffer(const std::string & name_, NamesAndTypesListPtr columns_, Context & context_, StorageBuffer(const std::string & name_, NamesAndTypesListPtr columns_,
const NamesAndTypesList & materialized_columns_,
const NamesAndTypesList & alias_columns_,
const ColumnDefaults & column_defaults_,
Context & context_,
size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_, size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_,
const String & destination_database_, const String & destination_table_); const String & destination_database_, const String & destination_table_);

View File

@ -14,7 +14,6 @@
#include <DB/AggregateFunctions/AggregateFunctionArray.h> #include <DB/AggregateFunctions/AggregateFunctionArray.h>
#include <DB/AggregateFunctions/AggregateFunctionState.h> #include <DB/AggregateFunctions/AggregateFunctionState.h>
#include <DB/AggregateFunctions/AggregateFunctionMerge.h> #include <DB/AggregateFunctions/AggregateFunctionMerge.h>
#include <DB/AggregateFunctions/AggregateFunctionDebug.h>
#include <DB/AggregateFunctions/AggregateFunctionSequenceMatch.h> #include <DB/AggregateFunctions/AggregateFunctionSequenceMatch.h>
#include <DB/AggregateFunctions/AggregateFunctionsStatistics.h> #include <DB/AggregateFunctions/AggregateFunctionsStatistics.h>
@ -233,9 +232,7 @@ static IAggregateFunction * createAggregateFunctionArgMinMax(const String & name
AggregateFunctionPtr AggregateFunctionFactory::get(const String & name, const DataTypes & argument_types, int recursion_level) const AggregateFunctionPtr AggregateFunctionFactory::get(const String & name, const DataTypes & argument_types, int recursion_level) const
{ {
if (name == "debug") if (name == "count")
return new AggregateFunctionDebug;
else if (name == "count")
return new AggregateFunctionCount; return new AggregateFunctionCount;
else if (name == "any") else if (name == "any")
return createAggregateFunctionSingleValue<AggregateFunctionsSingleValue, AggregateFunctionAnyData>(name, argument_types); return createAggregateFunctionSingleValue<AggregateFunctionsSingleValue, AggregateFunctionAnyData>(name, argument_types);
@ -697,7 +694,6 @@ const AggregateFunctionFactory::FunctionNames & AggregateFunctionFactory::getFun
{ {
static FunctionNames names static FunctionNames names
{ {
"debug",
"count", "count",
"any", "any",
"anyLast", "anyLast",

View File

@ -263,10 +263,10 @@ private:
else else
format = config().getString("format", is_interactive ? "PrettyCompact" : "TabSeparated"); format = config().getString("format", is_interactive ? "PrettyCompact" : "TabSeparated");
format_max_block_size = config().getInt("format_max_block_size", DEFAULT_BLOCK_SIZE); format_max_block_size = config().getInt("format_max_block_size", context.getSettingsRef().max_block_size);
insert_format = "Values"; insert_format = "Values";
insert_format_max_block_size = config().getInt("insert_format_max_block_size", DEFAULT_INSERT_BLOCK_SIZE); insert_format_max_block_size = config().getInt("insert_format_max_block_size", context.getSettingsRef().max_insert_block_size);
connect(); connect();

View File

@ -0,0 +1,71 @@
#include <vector>
#include <string>
#include <iomanip>
#include <DB/Common/SipHash.h>
#include <DB/IO/ReadBufferFromFileDescriptor.h>
#include <DB/IO/ReadHelpers.h>
#include <statdaemons/Stopwatch.h>
/** Тестировать так:
*
* clickhouse-client --query="SELECT SearchPhrase AS k FROM test.hits WHERE k != ''" > phrases.tsv
* clickhouse-client --query="SELECT URL AS k FROM test.hits" > urls.tsv
* clickhouse-client --query="SELECT SearchPhrase AS k FROM test.hits" > phrases_with_empty.tsv
* clickhouse-client --query="SELECT Title AS k FROM test.hits" > titles.tsv
* clickhouse-client --query="SELECT PageCharset AS k FROM test.hits" > charset.tsv
*
* for i in {1..1000}; do ./sip_hash_perf < titles.tsv 2>&1 | grep Processed | grep -oP '\d+\.\d+ rows/sec'; done | awk '{ if ($1 > x) { x = $1; print x } }'
*/
int main(int argc, char ** argv)
{
std::vector<std::string> data;
DB::ReadBufferFromFileDescriptor in(STDIN_FILENO);
std::cerr << std::fixed << std::setprecision(3);
{
Stopwatch watch;
while (!in.eof())
{
data.emplace_back();
DB::readEscapedString(data.back(), in);
DB::assertString("\n", in);
}
double seconds = watch.elapsedSeconds();
std::cerr << "Read "
<< data.size() << " rows, "
<< (in.count() / 1048576.0) << " MiB "
<< " in " << seconds << " sec., "
<< (data.size() / seconds) << " rows/sec., "
<< (in.count() / 1048576.0 / seconds) << " MiB/sec.\n";
}
{
size_t res = 0;
Stopwatch watch;
for (const auto & s : data)
{
SipHash hash;
hash.update(s.data(), s.size());
res += hash.get64();
}
double seconds = watch.elapsedSeconds();
std::cerr << "Processed "
<< data.size() << " rows, "
<< (in.count() / 1048576.0) << " MiB "
<< " in " << seconds << " sec., "
<< (data.size() / seconds) << " rows/sec., "
<< (in.count() / 1048576.0 / seconds) << " MiB/sec. "
<< "(res = " << res << ")\n";
}
return 0;
}

View File

@ -52,9 +52,6 @@ void BlockStreamProfileInfo::update(Block & block)
++blocks; ++blocks;
rows += block.rowsInFirstColumn(); rows += block.rowsInFirstColumn();
bytes += block.bytes(); bytes += block.bytes();
if (column_names.empty())
column_names = block.dumpNames();
} }

View File

@ -57,9 +57,9 @@ size_t IBlockInputStream::checkDepthImpl(size_t max_depth, size_t level) const
void IBlockInputStream::dumpTree(std::ostream & ostr, size_t indent, size_t multiplier) void IBlockInputStream::dumpTree(std::ostream & ostr, size_t indent, size_t multiplier)
{ {
/// Не будем отображать в дереве обёртку потока блоков в AsynchronousBlockInputStream. /// Не будем отображать в дереве обёртку потока блоков в AsynchronousBlockInputStream.
if (getShortName() != "Asynchronous") if (getName() != "Asynchronous")
{ {
ostr << String(indent, ' ') << getShortName(); ostr << String(indent, ' ') << getName();
if (multiplier > 1) if (multiplier > 1)
ostr << " × " << multiplier; ostr << " × " << multiplier;
ostr << std::endl; ostr << std::endl;
@ -91,15 +91,6 @@ void IBlockInputStream::dumpTree(std::ostream & ostr, size_t indent, size_t mult
} }
String IBlockInputStream::getShortName() const
{
String res = getName();
if (0 == strcmp(res.c_str() + res.size() - strlen("BlockInputStream"), "BlockInputStream"))
res = res.substr(0, res.size() - strlen("BlockInputStream"));
return res;
}
BlockInputStreams IBlockInputStream::getLeaves() BlockInputStreams IBlockInputStream::getLeaves()
{ {
BlockInputStreams res; BlockInputStreams res;

View File

@ -19,7 +19,7 @@ Block IProfilingBlockInputStream::read()
if (!info.started) if (!info.started)
{ {
info.total_stopwatch.start(); info.total_stopwatch.start();
info.stream_name = getShortName(); info.stream_name = getName();
for (const auto & child : children) for (const auto & child : children)
if (const IProfilingBlockInputStream * p_child = dynamic_cast<const IProfilingBlockInputStream *>(&*child)) if (const IProfilingBlockInputStream * p_child = dynamic_cast<const IProfilingBlockInputStream *>(&*child))
@ -43,7 +43,7 @@ Block IProfilingBlockInputStream::read()
Poco::ScopedLock<Poco::FastMutex> lock(mutex); Poco::ScopedLock<Poco::FastMutex> lock(mutex);
std::cerr << std::endl; std::cerr << std::endl;
std::cerr << "[ " << Poco::ThreadNumber::get() << " ]\t" << getShortName() << std::endl; std::cerr << "[ " << Poco::ThreadNumber::get() << " ]\t" << getName() << std::endl;
std::cerr << "[ " << Poco::ThreadNumber::get() << " ]\t"; std::cerr << "[ " << Poco::ThreadNumber::get() << " ]\t";
for (size_t i = 0; i < res.columns(); ++i) for (size_t i = 0; i < res.columns(); ++i)

View File

@ -340,6 +340,9 @@ void registerFunctionsMiscellaneous(FunctionFactory & factory)
factory.registerFunction<FunctionIsNaN>(); factory.registerFunction<FunctionIsNaN>();
factory.registerFunction<FunctionVersion>(); factory.registerFunction<FunctionVersion>();
factory.registerFunction<FunctionRunningAccumulate>();
factory.registerFunction<FunctionFinalizeAggregation>();
} }
} }

View File

@ -20,10 +20,6 @@ void registerFunctionsString(FunctionFactory & factory)
factory.registerFunction<FunctionSubstring>(); factory.registerFunction<FunctionSubstring>();
factory.registerFunction<FunctionSubstringUTF8>(); factory.registerFunction<FunctionSubstringUTF8>();
factory.registerFunction<FunctionAppendTrailingCharIfAbsent>(); factory.registerFunction<FunctionAppendTrailingCharIfAbsent>();
factory.registerFunction<FunctionSSELower>();
factory.registerFunction<FunctionSSEUpper>();
factory.registerFunction<FunctionSSELowerUTF8>();
factory.registerFunction<FunctionSSEUpperUTF8>();
} }
} }

View File

@ -124,12 +124,13 @@ void ExpressionAction::prepare(Block & sample_block)
/// Если все аргументы и требуемые столбцы - константы, выполним функцию. /// Если все аргументы и требуемые столбцы - константы, выполним функцию.
if (all_const) if (all_const)
{ {
size_t result_position = sample_block.columns();
ColumnWithNameAndType new_column; ColumnWithNameAndType new_column;
new_column.name = result_name; new_column.name = result_name;
new_column.type = result_type; new_column.type = result_type;
sample_block.insert(new_column); sample_block.insert(new_column);
size_t result_position = sample_block.getPositionByName(result_name);
function->execute(sample_block, arguments, prerequisites, result_position); function->execute(sample_block, arguments, prerequisites, result_position);
/// Если получилась не константа, на всякий случай будем считать результат неизвестным. /// Если получилась не константа, на всякий случай будем считать результат неизвестным.

View File

@ -922,16 +922,24 @@ void ExpressionAnalyzer::makeExplicitSet(ASTFunction * node, const Block & sampl
if (ASTFunction * set_func = typeid_cast<ASTFunction *>(&*arg)) if (ASTFunction * set_func = typeid_cast<ASTFunction *>(&*arg))
{ {
if (set_func->name != "tuple") if (set_func->name == "tuple")
throw Exception("Incorrect type of 2nd argument for function " + node->name + ". Must be subquery or set of values.", {
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT); /// Отличм случай (x, y) in ((1, 2), (3, 4)) от случая (x, y) in (1, 2).
ASTFunction * any_element = typeid_cast<ASTFunction *>(&*set_func->arguments->children.at(0));
/// Отличм случай (x, y) in ((1, 2), (3, 4)) от случая (x, y) in (1, 2). if (set_element_types.size() >= 2 && (!any_element || any_element->name != "tuple"))
ASTFunction * any_element = typeid_cast<ASTFunction *>(&*set_func->arguments->children.at(0)); single_value = true;
if (set_element_types.size() >= 2 && (!any_element || any_element->name != "tuple")) else
single_value = true; elements_ast = set_func->arguments;
}
else else
elements_ast = set_func->arguments; {
if (set_element_types.size() >= 2)
throw Exception("Incorrect type of 2nd argument for function " + node->name
+ ". Must be subquery or set of " + toString(set_element_types.size()) + "-element tuples.",
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
single_value = true;
}
} }
else if (typeid_cast<ASTLiteral *>(&*arg)) else if (typeid_cast<ASTLiteral *>(&*arg))
{ {
@ -954,7 +962,7 @@ void ExpressionAnalyzer::makeExplicitSet(ASTFunction * node, const Block & sampl
ASTPtr ast_set_ptr = ast_set; ASTPtr ast_set_ptr = ast_set;
ast_set->set = new Set(settings.limits); ast_set->set = new Set(settings.limits);
ast_set->is_explicit = true; ast_set->is_explicit = true;
ast_set->set->createFromAST(set_element_types, elements_ast, create_ordered_set); ast_set->set->createFromAST(set_element_types, elements_ast, context, create_ordered_set);
arg = ast_set_ptr; arg = ast_set_ptr;
} }
@ -1082,13 +1090,11 @@ void ExpressionAnalyzer::getArrayJoinedColumns()
if (select_query && select_query->array_join_expression_list) if (select_query && select_query->array_join_expression_list)
{ {
ASTs & array_join_asts = select_query->array_join_expression_list->children; ASTs & array_join_asts = select_query->array_join_expression_list->children;
for (size_t i = 0; i < array_join_asts .size(); ++i) for (const auto & ast : array_join_asts)
{ {
ASTPtr ast = array_join_asts [i]; const String nested_table_name = ast->getColumnName();
const String nested_table_alias = ast->getAliasOrColumnName();
String nested_table_name = ast->getColumnName(); if (nested_table_alias == nested_table_name && !typeid_cast<const ASTIdentifier *>(&*ast))
String nested_table_alias = ast->getAliasOrColumnName();
if (nested_table_alias == nested_table_name && !typeid_cast<ASTIdentifier *>(&*ast))
throw Exception("No alias for non-trivial value in ARRAY JOIN: " + nested_table_name, ErrorCodes::ALIAS_REQUIRED); throw Exception("No alias for non-trivial value in ARRAY JOIN: " + nested_table_name, ErrorCodes::ALIAS_REQUIRED);
if (array_join_alias_to_name.count(nested_table_alias) || aliases.count(nested_table_alias)) if (array_join_alias_to_name.count(nested_table_alias) || aliases.count(nested_table_alias))
@ -1097,13 +1103,9 @@ void ExpressionAnalyzer::getArrayJoinedColumns()
} }
ASTs & query_asts = select_query->children; ASTs & query_asts = select_query->children;
for (size_t i = 0; i < query_asts.size(); ++i) for (const auto & ast : query_asts)
{ if (ast != select_query->array_join_expression_list)
ASTPtr ast = query_asts[i]; getArrayJoinedColumnsImpl(ast);
if (select_query && ast == select_query->array_join_expression_list)
continue;
getArrayJoinedColumnsImpl(ast);
}
/// Если результат ARRAY JOIN не используется, придется все равно по-ARRAY-JOIN-ить какой-нибудь столбец, /// Если результат ARRAY JOIN не используется, придется все равно по-ARRAY-JOIN-ить какой-нибудь столбец,
/// чтобы получить правильное количество строк. /// чтобы получить правильное количество строк.

View File

@ -6,6 +6,15 @@
#include <Poco/Util/Application.h> #include <Poco/Util/Application.h>
#include <Poco/Glob.h> #include <Poco/Glob.h>
namespace
{
/// 5 seconds
const auto backoff_initial_sec = 5;
/// 10 minutes
const auto backoff_max_sec = 10 * 60;
}
namespace DB namespace DB
{ {
@ -41,6 +50,63 @@ void ExternalDictionaries::reloadImpl(const bool throw_on_error)
for (const auto & config_path : config_paths) for (const auto & config_path : config_paths)
reloadFromFile(config_path, throw_on_error); reloadFromFile(config_path, throw_on_error);
/// list of recreated dictionaries to perform delayed removal from unordered_map
std::list<std::string> recreated_failed_dictionaries;
/// retry loading failed dictionaries
for (auto & failed_dictionary : failed_dictionaries)
{
if (std::chrono::system_clock::now() < failed_dictionary.second.next_attempt_time)
continue;
const auto & name = failed_dictionary.first;
try
{
auto dict_ptr = failed_dictionary.second.dict->clone();
if (dict_ptr->getCreationException())
{
/// recalculate next attempt time
std::uniform_int_distribution<std::uint64_t> distribution(
0, std::exp2(failed_dictionary.second.error_count));
failed_dictionary.second.next_attempt_time = std::chrono::system_clock::now() +
std::chrono::seconds{
std::min<std::uint64_t>(backoff_max_sec, backoff_initial_sec + distribution(rnd_engine))
};
++failed_dictionary.second.error_count;
}
else
{
const std::lock_guard<std::mutex> lock{dictionaries_mutex};
const auto dict_it = dictionaries.find(name);
if (dict_it->second.dict)
dict_it->second.dict->set(dict_ptr.release());
else
dict_it->second.dict = std::make_shared<MultiVersion<IDictionary>>(dict_ptr.release());
/// erase stored exception on success
dict_it->second.exception = std::exception_ptr{};
const auto & lifetime = dict_ptr->getLifetime();
std::uniform_int_distribution<std::uint64_t> distribution{lifetime.min_sec, lifetime.max_sec};
update_times[name] = std::chrono::system_clock::now() + std::chrono::seconds{distribution(rnd_engine)};
recreated_failed_dictionaries.push_back(name);
}
}
catch (...)
{
LOG_ERROR(log, "Failed reloading " << name << " dictionary due to unexpected error");
}
}
/// do not undertake further attempts to recreate these dictionaries
for (const auto & name : recreated_failed_dictionaries)
failed_dictionaries.erase(name);
/// periodic update /// periodic update
for (auto & dictionary : dictionaries) for (auto & dictionary : dictionaries)
{ {
@ -122,10 +188,10 @@ void ExternalDictionaries::reloadFromFile(const std::string & config_path, const
} }
else else
{ {
auto it = last_modification_times.find(config_path); auto modification_time_it = last_modification_times.find(config_path);
if (it == std::end(last_modification_times)) if (modification_time_it == std::end(last_modification_times))
it = last_modification_times.emplace(config_path, Poco::Timestamp{0}).first; modification_time_it = last_modification_times.emplace(config_path, Poco::Timestamp{0}).first;
auto & config_last_modified = it->second; auto & config_last_modified = modification_time_it->second;
const auto last_modified = config_file.getLastModified(); const auto last_modified = config_file.getLastModified();
if (last_modified > config_last_modified) if (last_modified > config_last_modified)
@ -163,12 +229,31 @@ void ExternalDictionaries::reloadFromFile(const std::string & config_path, const
continue; continue;
} }
auto it = dictionaries.find(name); const auto dict_it = dictionaries.find(name);
if (it != std::end(dictionaries)) if (dict_it != std::end(dictionaries))
if (it->second.origin != config_path) if (dict_it->second.origin != config_path)
throw std::runtime_error{"Overriding dictionary from file " + it->second.origin}; throw std::runtime_error{"Overriding dictionary from file " + dict_it->second.origin};
auto dict_ptr = DictionaryFactory::instance().create(name, *config, key, context); auto dict_ptr = DictionaryFactory::instance().create(name, *config, key, context);
if (const auto exception_ptr = dict_ptr->getCreationException())
{
const auto failed_dict_it = failed_dictionaries.find(name);
if (failed_dict_it != std::end(failed_dictionaries))
{
failed_dict_it->second = failed_dictionary_info{
std::move(dict_ptr),
std::chrono::system_clock::now() + std::chrono::seconds{backoff_initial_sec}
};
}
else
failed_dictionaries.emplace(name, failed_dictionary_info{
std::move(dict_ptr),
std::chrono::system_clock::now() + std::chrono::seconds{backoff_initial_sec}
});
std::rethrow_exception(exception_ptr);
}
if (!dict_ptr->isCached()) if (!dict_ptr->isCached())
{ {
const auto & lifetime = dict_ptr->getLifetime(); const auto & lifetime = dict_ptr->getLifetime();
@ -183,42 +268,38 @@ void ExternalDictionaries::reloadFromFile(const std::string & config_path, const
} }
} }
const std::lock_guard<std::mutex> lock{dictionaries_mutex};
/// add new dictionary or update an existing version /// add new dictionary or update an existing version
if (it == std::end(dictionaries)) if (dict_it == std::end(dictionaries))
{
const std::lock_guard<std::mutex> lock{dictionaries_mutex};
dictionaries.emplace(name, dictionary_info{ dictionaries.emplace(name, dictionary_info{
std::make_shared<MultiVersion<IDictionary>>(dict_ptr.release()), std::make_shared<MultiVersion<IDictionary>>(dict_ptr.release()),
config_path config_path
}); });
}
else else
{ {
if (it->second.dict) if (dict_it->second.dict)
it->second.dict->set(dict_ptr.release()); dict_it->second.dict->set(dict_ptr.release());
else else
{ dict_it->second.dict = std::make_shared<MultiVersion<IDictionary>>(dict_ptr.release());
const std::lock_guard<std::mutex> lock{dictionaries_mutex};
it->second.dict = std::make_shared<MultiVersion<IDictionary>>(dict_ptr.release());
}
/// erase stored exception on success /// erase stored exception on success
it->second.exception = std::exception_ptr{}; dict_it->second.exception = std::exception_ptr{};
failed_dictionaries.erase(name);
} }
} }
catch (...) catch (...)
{ {
if (!name.empty()) if (!name.empty())
{ {
const std::lock_guard<std::mutex> lock{dictionaries_mutex};
const auto exception_ptr = std::current_exception(); const auto exception_ptr = std::current_exception();
const auto it = dictionaries.find(name); const auto dict_it = dictionaries.find(name);
if (it == std::end(dictionaries)) if (dict_it == std::end(dictionaries))
{
const std::lock_guard<std::mutex> lock{dictionaries_mutex};
dictionaries.emplace(name, dictionary_info{nullptr, config_path, exception_ptr}); dictionaries.emplace(name, dictionary_info{nullptr, config_path, exception_ptr});
}
else else
it->second.exception = exception_ptr; dict_it->second.exception = exception_ptr;
} }
try try
@ -253,16 +334,17 @@ void ExternalDictionaries::reloadFromFile(const std::string & config_path, const
MultiVersion<IDictionary>::Version ExternalDictionaries::getDictionary(const std::string & name) const MultiVersion<IDictionary>::Version ExternalDictionaries::getDictionary(const std::string & name) const
{ {
const std::lock_guard<std::mutex> lock{dictionaries_mutex}; const std::lock_guard<std::mutex> lock{dictionaries_mutex};
const auto it = dictionaries.find(name);
const auto it = dictionaries.find(name);
if (it == std::end(dictionaries)) if (it == std::end(dictionaries))
throw Exception{ throw Exception{
"No such dictionary: " + name, "No such dictionary: " + name,
ErrorCodes::BAD_ARGUMENTS ErrorCodes::BAD_ARGUMENTS
}; };
if (!it->second.dict && it->second.exception) if (!it->second.dict)
std::rethrow_exception(it->second.exception); it->second.exception ? std::rethrow_exception(it->second.exception) :
throw Exception{"No dictionary", ErrorCodes::LOGICAL_ERROR};
return it->second.dict->get(); return it->second.dict->get();
} }

View File

@ -12,7 +12,6 @@ InterpreterCheckQuery::InterpreterCheckQuery(DB::ASTPtr query_ptr_, DB::Context&
BlockInputStreamPtr InterpreterCheckQuery::execute() BlockInputStreamPtr InterpreterCheckQuery::execute()
{ {
/// @TODO
ASTCheckQuery & alter = typeid_cast<ASTCheckQuery &>(*query_ptr); ASTCheckQuery & alter = typeid_cast<ASTCheckQuery &>(*query_ptr);
String & table_name = alter.table; String & table_name = alter.table;
String database_name = alter.database.empty() ? context.getCurrentDatabase() : alter.database; String database_name = alter.database.empty() ? context.getCurrentDatabase() : alter.database;

View File

@ -24,6 +24,7 @@
#include <DB/Parsers/ASTOrderByElement.h> #include <DB/Parsers/ASTOrderByElement.h>
#include <DB/Interpreters/InterpreterSelectQuery.h> #include <DB/Interpreters/InterpreterSelectQuery.h>
#include <DB/Interpreters/InterpreterSetQuery.h>
#include <DB/Interpreters/ExpressionAnalyzer.h> #include <DB/Interpreters/ExpressionAnalyzer.h>
#include <DB/Storages/StorageView.h> #include <DB/Storages/StorageView.h>
#include <DB/TableFunctions/ITableFunction.h> #include <DB/TableFunctions/ITableFunction.h>
@ -39,10 +40,12 @@ InterpreterSelectQuery::~InterpreterSelectQuery() = default;
void InterpreterSelectQuery::init(BlockInputStreamPtr input, const Names & required_column_names, const NamesAndTypesList & table_column_names) void InterpreterSelectQuery::init(BlockInputStreamPtr input, const Names & required_column_names, const NamesAndTypesList & table_column_names)
{ {
original_max_threads = settings.max_threads;
ProfileEvents::increment(ProfileEvents::SelectQuery); ProfileEvents::increment(ProfileEvents::SelectQuery);
initSettings();
original_max_threads = settings.max_threads;
if (settings.limits.max_subquery_depth && subquery_depth > settings.limits.max_subquery_depth) if (settings.limits.max_subquery_depth && subquery_depth > settings.limits.max_subquery_depth)
throw Exception("Too deep subqueries. Maximum: " + toString(settings.limits.max_subquery_depth), throw Exception("Too deep subqueries. Maximum: " + toString(settings.limits.max_subquery_depth),
ErrorCodes::TOO_DEEP_SUBQUERIES); ErrorCodes::TOO_DEEP_SUBQUERIES);
@ -174,7 +177,7 @@ void InterpreterSelectQuery::initQueryAnalyzer()
InterpreterSelectQuery::InterpreterSelectQuery(ASTPtr query_ptr_, const Context & context_, QueryProcessingStage::Enum to_stage_, InterpreterSelectQuery::InterpreterSelectQuery(ASTPtr query_ptr_, const Context & context_, QueryProcessingStage::Enum to_stage_,
size_t subquery_depth_, BlockInputStreamPtr input_, bool is_union_all_head_) size_t subquery_depth_, BlockInputStreamPtr input_, bool is_union_all_head_)
: query_ptr(query_ptr_), query(typeid_cast<ASTSelectQuery &>(*query_ptr)), : query_ptr(query_ptr_), query(typeid_cast<ASTSelectQuery &>(*query_ptr)),
context(context_), settings(context.getSettings()), to_stage(to_stage_), subquery_depth(subquery_depth_), context(context_), to_stage(to_stage_), subquery_depth(subquery_depth_),
is_first_select_inside_union_all(is_union_all_head_ && !query.next_union_all.isNull()), is_first_select_inside_union_all(is_union_all_head_ && !query.next_union_all.isNull()),
log(&Logger::get("InterpreterSelectQuery")) log(&Logger::get("InterpreterSelectQuery"))
{ {
@ -185,7 +188,7 @@ InterpreterSelectQuery::InterpreterSelectQuery(ASTPtr query_ptr_, const Context
const Names & required_column_names_, const Names & required_column_names_,
QueryProcessingStage::Enum to_stage_, size_t subquery_depth_, BlockInputStreamPtr input_) QueryProcessingStage::Enum to_stage_, size_t subquery_depth_, BlockInputStreamPtr input_)
: query_ptr(query_ptr_), query(typeid_cast<ASTSelectQuery &>(*query_ptr)), : query_ptr(query_ptr_), query(typeid_cast<ASTSelectQuery &>(*query_ptr)),
context(context_), settings(context.getSettings()), to_stage(to_stage_), subquery_depth(subquery_depth_), context(context_), to_stage(to_stage_), subquery_depth(subquery_depth_),
is_first_select_inside_union_all(!query.next_union_all.isNull()), is_first_select_inside_union_all(!query.next_union_all.isNull()),
log(&Logger::get("InterpreterSelectQuery")) log(&Logger::get("InterpreterSelectQuery"))
{ {
@ -196,7 +199,7 @@ InterpreterSelectQuery::InterpreterSelectQuery(ASTPtr query_ptr_, const Context
const Names & required_column_names_, const Names & required_column_names_,
const NamesAndTypesList & table_column_names, QueryProcessingStage::Enum to_stage_, size_t subquery_depth_, BlockInputStreamPtr input_) const NamesAndTypesList & table_column_names, QueryProcessingStage::Enum to_stage_, size_t subquery_depth_, BlockInputStreamPtr input_)
: query_ptr(query_ptr_), query(typeid_cast<ASTSelectQuery &>(*query_ptr)), : query_ptr(query_ptr_), query(typeid_cast<ASTSelectQuery &>(*query_ptr)),
context(context_), settings(context.getSettings()), to_stage(to_stage_), subquery_depth(subquery_depth_), context(context_), to_stage(to_stage_), subquery_depth(subquery_depth_),
is_first_select_inside_union_all(!query.next_union_all.isNull()), is_first_select_inside_union_all(!query.next_union_all.isNull()),
log(&Logger::get("InterpreterSelectQuery")) log(&Logger::get("InterpreterSelectQuery"))
{ {
@ -1028,4 +1031,12 @@ void InterpreterSelectQuery::ignoreWithTotals()
} }
void InterpreterSelectQuery::initSettings()
{
if (query.settings)
InterpreterSetQuery(query.settings, context).executeForCurrentContext();
settings = context.getSettings();
}
} }

View File

@ -339,15 +339,19 @@ void Join::setSampleBlock(const Block & block)
/// Выберем, какую структуру данных для множества использовать. /// Выберем, какую структуру данных для множества использовать.
init(chooseMethod(key_columns, keys_fit_128_bits, key_sizes)); init(chooseMethod(key_columns, keys_fit_128_bits, key_sizes));
sample_block = block; sample_block_with_columns_to_add = block;
/// Удаляем из sample_block ключевые столбцы, так как они не нужны. /// Удаляем из sample_block_with_columns_to_add ключевые столбцы.
for (const auto & name : key_names_right) for (const auto & name : key_names_right)
sample_block.erase(sample_block.getPositionByName(name));
for (size_t i = 0, size = sample_block.columns(); i < size; ++i)
{ {
auto & column = sample_block.unsafeGetByPosition(i); size_t pos = sample_block_with_columns_to_add.getPositionByName(name);
sample_block_with_keys.insert(sample_block_with_columns_to_add.unsafeGetByPosition(pos));
sample_block_with_columns_to_add.erase(pos);
}
for (size_t i = 0, size = sample_block_with_columns_to_add.columns(); i < size; ++i)
{
auto & column = sample_block_with_columns_to_add.unsafeGetByPosition(i);
if (!column.column) if (!column.column)
column.column = column.type->createColumn(); column.column = column.type->createColumn();
} }
@ -365,10 +369,21 @@ bool Join::insertFromBlock(const Block & block)
size_t keys_size = key_names_right.size(); size_t keys_size = key_names_right.size();
ConstColumnPlainPtrs key_columns(keys_size); ConstColumnPlainPtrs key_columns(keys_size);
/// Редкий случай, когда ключи являются константами. Чтобы не поддерживать отдельный код, материализуем их.
Columns materialized_columns;
/// Запоминаем столбцы ключей, с которыми будем работать /// Запоминаем столбцы ключей, с которыми будем работать
for (size_t i = 0; i < keys_size; ++i) for (size_t i = 0; i < keys_size; ++i)
{
key_columns[i] = block.getByName(key_names_right[i]).column; key_columns[i] = block.getByName(key_names_right[i]).column;
if (key_columns[i]->isConst())
{
materialized_columns.emplace_back(dynamic_cast<const IColumnConst &>(*key_columns[i]).convertToFullColumn());
key_columns[i] = materialized_columns.back();
}
}
size_t rows = block.rows(); size_t rows = block.rows();
blocks.push_back(block); blocks.push_back(block);
@ -378,7 +393,7 @@ bool Join::insertFromBlock(const Block & block)
for (const auto & name : key_names_right) for (const auto & name : key_names_right)
stored_block->erase(stored_block->getPositionByName(name)); stored_block->erase(stored_block->getPositionByName(name));
/// Редкий случай, когда соединяемые столбцы являеются константами. Чтобы не поддерживать отдельный код, материализуем их. /// Редкий случай, когда соединяемые столбцы являются константами. Чтобы не поддерживать отдельный код, материализуем их.
for (size_t i = 0, size = stored_block->columns(); i < size; ++i) for (size_t i = 0, size = stored_block->columns(); i < size; ++i)
{ {
ColumnPtr col = stored_block->getByPosition(i).column; ColumnPtr col = stored_block->getByPosition(i).column;
@ -515,19 +530,30 @@ void Join::joinBlockImpl(Block & block, const Maps & maps) const
size_t keys_size = key_names_left.size(); size_t keys_size = key_names_left.size();
ConstColumnPlainPtrs key_columns(keys_size); ConstColumnPlainPtrs key_columns(keys_size);
/// Редкий случай, когда ключи являются константами. Чтобы не поддерживать отдельный код, материализуем их.
Columns materialized_columns;
/// Запоминаем столбцы ключей, с которыми будем работать /// Запоминаем столбцы ключей, с которыми будем работать
for (size_t i = 0; i < keys_size; ++i) for (size_t i = 0; i < keys_size; ++i)
{
key_columns[i] = block.getByName(key_names_left[i]).column; key_columns[i] = block.getByName(key_names_left[i]).column;
if (key_columns[i]->isConst())
{
materialized_columns.emplace_back(dynamic_cast<const IColumnConst &>(*key_columns[i]).convertToFullColumn());
key_columns[i] = materialized_columns.back();
}
}
/// Добавляем в блок новые столбцы. /// Добавляем в блок новые столбцы.
size_t num_columns_to_add = sample_block.columns(); size_t num_columns_to_add = sample_block_with_columns_to_add.columns();
ColumnPlainPtrs added_columns(num_columns_to_add); ColumnPlainPtrs added_columns(num_columns_to_add);
size_t existing_columns = block.columns(); size_t existing_columns = block.columns();
for (size_t i = 0; i < num_columns_to_add; ++i) for (size_t i = 0; i < num_columns_to_add; ++i)
{ {
const ColumnWithNameAndType & src_column = sample_block.getByPosition(i); const ColumnWithNameAndType & src_column = sample_block_with_columns_to_add.getByPosition(i);
ColumnWithNameAndType new_column = src_column.cloneEmpty(); ColumnWithNameAndType new_column = src_column.cloneEmpty();
block.insert(new_column); block.insert(new_column);
added_columns[i] = new_column.column; added_columns[i] = new_column.column;
@ -628,10 +654,25 @@ void Join::joinBlockImpl(Block & block, const Maps & maps) const
} }
void Join::checkTypesOfKeys(const Block & block_left, const Block & block_right) const
{
size_t keys_size = key_names_left.size();
for (size_t i = 0; i < keys_size; ++i)
if (block_left.getByName(key_names_left[i]).type->getName() != block_right.getByName(key_names_right[i]).type->getName())
throw Exception("Type mismatch of columns to JOIN by: "
+ key_names_left[i] + " " + block_left.getByName(key_names_left[i]).type->getName() + " at left, "
+ key_names_right[i] + " " + block_right.getByName(key_names_right[i]).type->getName() + " at right",
ErrorCodes::TYPE_MISMATCH);
}
void Join::joinBlock(Block & block) const void Join::joinBlock(Block & block) const
{ {
Poco::ScopedReadRWLock lock(rwlock); Poco::ScopedReadRWLock lock(rwlock);
checkTypesOfKeys(block, sample_block_with_keys);
if (kind == ASTJoin::Left && strictness == ASTJoin::Any) if (kind == ASTJoin::Left && strictness == ASTJoin::Any)
joinBlockImpl<ASTJoin::Left, ASTJoin::Any>(block, maps_any); joinBlockImpl<ASTJoin::Left, ASTJoin::Any>(block, maps_any);
else if (kind == ASTJoin::Inner && strictness == ASTJoin::Any) else if (kind == ASTJoin::Inner && strictness == ASTJoin::Any)
@ -666,7 +707,7 @@ void Join::joinTotals(Block & block) const
else else
{ {
/// Будем присоединять пустые totals - из одной строчки со значениями по-умолчанию. /// Будем присоединять пустые totals - из одной строчки со значениями по-умолчанию.
totals_without_keys = sample_block.cloneEmpty(); totals_without_keys = sample_block_with_columns_to_add.cloneEmpty();
for (size_t i = 0; i < totals_without_keys.columns(); ++i) for (size_t i = 0; i < totals_without_keys.columns(); ++i)
{ {
@ -723,7 +764,7 @@ public:
{ {
} }
String getName() const override { return "NonJoinedBlockInputStream"; } String getName() const override { return "NonJoined"; }
String getID() const override String getID() const override
{ {
@ -771,12 +812,12 @@ private:
} }
/// Добавляем в блок новые столбцы. /// Добавляем в блок новые столбцы.
size_t num_columns_right = parent.sample_block.columns(); size_t num_columns_right = parent.sample_block_with_columns_to_add.columns();
ColumnPlainPtrs columns_right(num_columns_right); ColumnPlainPtrs columns_right(num_columns_right);
for (size_t i = 0; i < num_columns_right; ++i) for (size_t i = 0; i < num_columns_right; ++i)
{ {
const ColumnWithNameAndType & src_column = parent.sample_block.getByPosition(i); const ColumnWithNameAndType & src_column = parent.sample_block_with_columns_to_add.getByPosition(i);
ColumnWithNameAndType new_column = src_column.cloneEmpty(); ColumnWithNameAndType new_column = src_column.cloneEmpty();
block.insert(new_column); block.insert(new_column);
columns_right[i] = new_column.column; columns_right[i] = new_column.column;

View File

@ -12,10 +12,15 @@
#include <DB/Parsers/ASTLiteral.h> #include <DB/Parsers/ASTLiteral.h>
#include <DB/Interpreters/Set.h> #include <DB/Interpreters/Set.h>
#include <DB/Interpreters/ExpressionAnalyzer.h>
#include <DB/Interpreters/ExpressionActions.h>
#include <DB/DataTypes/DataTypeArray.h> #include <DB/DataTypes/DataTypeArray.h>
#include <DB/DataTypes/DataTypesNumberFixed.h> #include <DB/DataTypes/DataTypesNumberFixed.h>
#include <DB/DataTypes/DataTypeString.h> #include <DB/DataTypes/DataTypeString.h>
#include <DB/DataTypes/DataTypeFixedString.h> #include <DB/DataTypes/DataTypeFixedString.h>
#include <DB/DataTypes/DataTypeDate.h>
#include <DB/DataTypes/DataTypeDateTime.h>
namespace DB namespace DB
@ -231,7 +236,7 @@ bool Set::insertFromBlock(const Block & block, bool create_ordered_set)
*/ */
static Field convertToType(const Field & src, const IDataType & type) static Field convertToType(const Field & src, const IDataType & type)
{ {
if (type.behavesAsNumber()) if (type.isNumeric())
{ {
bool is_uint8 = false; bool is_uint8 = false;
bool is_uint16 = false; bool is_uint16 = false;
@ -243,6 +248,8 @@ static Field convertToType(const Field & src, const IDataType & type)
bool is_int64 = false; bool is_int64 = false;
bool is_float32 = false; bool is_float32 = false;
bool is_float64 = false; bool is_float64 = false;
bool is_date = false;
bool is_datetime = false;
false false
|| (is_uint8 = typeid_cast<const DataTypeUInt8 * >(&type)) || (is_uint8 = typeid_cast<const DataTypeUInt8 * >(&type))
@ -254,15 +261,18 @@ static Field convertToType(const Field & src, const IDataType & type)
|| (is_int32 = typeid_cast<const DataTypeInt32 * >(&type)) || (is_int32 = typeid_cast<const DataTypeInt32 * >(&type))
|| (is_int64 = typeid_cast<const DataTypeInt64 * >(&type)) || (is_int64 = typeid_cast<const DataTypeInt64 * >(&type))
|| (is_float32 = typeid_cast<const DataTypeFloat32 * >(&type)) || (is_float32 = typeid_cast<const DataTypeFloat32 * >(&type))
|| (is_float64 = typeid_cast<const DataTypeFloat64 * >(&type)); || (is_float64 = typeid_cast<const DataTypeFloat64 * >(&type))
|| (is_date = typeid_cast<const DataTypeDate * >(&type))
|| (is_datetime = typeid_cast<const DataTypeDateTime * >(&type))
;
if (is_uint8 || is_uint16 || is_uint32 || is_uint64) if (is_uint8 || is_uint16 || is_uint32 || is_uint64)
{ {
if (src.getType() == Field::Types::Int64) if (src.getType() == Field::Types::Int64)
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, signed literal at right"); throw Exception("Type mismatch in IN section: " + type.getName() + " at left, signed at right");
if (src.getType() == Field::Types::Float64) if (src.getType() == Field::Types::Float64)
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, floating point literal at right"); throw Exception("Type mismatch in IN section: " + type.getName() + " at left, floating point at right");
if (src.getType() == Field::Types::UInt64) if (src.getType() == Field::Types::UInt64)
{ {
@ -276,12 +286,12 @@ static Field convertToType(const Field & src, const IDataType & type)
} }
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, " throw Exception("Type mismatch in IN section: " + type.getName() + " at left, "
+ Field::Types::toString(src.getType()) + " literal at right"); + Field::Types::toString(src.getType()) + " at right");
} }
else if (is_int8 || is_int16 || is_int32 || is_int64) else if (is_int8 || is_int16 || is_int32 || is_int64)
{ {
if (src.getType() == Field::Types::Float64) if (src.getType() == Field::Types::Float64)
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, floating point literal at right"); throw Exception("Type mismatch in IN section: " + type.getName() + " at left, floating point at right");
if (src.getType() == Field::Types::UInt64) if (src.getType() == Field::Types::UInt64)
{ {
@ -308,7 +318,7 @@ static Field convertToType(const Field & src, const IDataType & type)
} }
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, " throw Exception("Type mismatch in IN section: " + type.getName() + " at left, "
+ Field::Types::toString(src.getType()) + " literal at right"); + Field::Types::toString(src.getType()) + " at right");
} }
else if (is_float32 || is_float64) else if (is_float32 || is_float64)
{ {
@ -322,24 +332,77 @@ static Field convertToType(const Field & src, const IDataType & type)
return src; return src;
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, " throw Exception("Type mismatch in IN section: " + type.getName() + " at left, "
+ Field::Types::toString(src.getType()) + " literal at right"); + Field::Types::toString(src.getType()) + " at right");
}
else if (is_date || is_datetime)
{
if (src.getType() != Field::Types::UInt64)
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, "
+ Field::Types::toString(src.getType()) + " at right");
return src;
} }
} }
else
{
if (src.getType() == Field::Types::UInt64
|| src.getType() == Field::Types::Int64
|| src.getType() == Field::Types::Float64
|| src.getType() == Field::Types::Null
|| (src.getType() == Field::Types::String
&& !typeid_cast<const DataTypeString *>(&type)
&& !typeid_cast<const DataTypeFixedString *>(&type))
|| (src.getType() == Field::Types::Array
&& !typeid_cast<const DataTypeArray *>(&type)))
throw Exception("Type mismatch in IN section: " + type.getName() + " at left, "
+ Field::Types::toString(src.getType()) + " at right");
}
/// В остальных случаях, приведение типа не осуществляется.
return src; return src;
} }
void Set::createFromAST(DataTypes & types, ASTPtr node, bool create_ordered_set) /** Выполнить константное выражение (для элемента множества в IN). Весьма неоптимально. */
static Field evaluateConstantExpression(ASTPtr & node, const Context & context)
{ {
/** NOTE: ExpressionActionsPtr expr_for_constant_folding = ExpressionAnalyzer(
* На данный момент в секции IN не поддерживаются выражения (вызовы функций), кроме кортежей. node, context, NamesAndTypesList{{ "_dummy", new DataTypeUInt8 }}).getConstActions();
* То есть, например, не поддерживаются массивы. А по хорошему, хотелось бы поддерживать.
* Для этого можно сделать constant folding с помощью ExpressionAnalyzer/ExpressionActions.
* Но при этом, конечно же, не забыть про производительность работы с крупными множествами.
*/
/// В блоке должен быть хотя бы один столбец, чтобы у него было известно число строк.
Block block_with_constants{{ new ColumnConstUInt8(1, 0), new DataTypeUInt8, "_dummy" }};
expr_for_constant_folding->execute(block_with_constants);
if (!block_with_constants || block_with_constants.rows() == 0)
throw Exception("Logical error: empty block after evaluation constant expression for IN", ErrorCodes::LOGICAL_ERROR);
String name = node->getColumnName();
if (!block_with_constants.has(name))
throw Exception("Element of set in IN is not a constant expression: " + name, ErrorCodes::BAD_ARGUMENTS);
const IColumn & result_column = *block_with_constants.getByName(name).column;
if (!result_column.isConst())
throw Exception("Element of set in IN is not a constant expression: " + name, ErrorCodes::BAD_ARGUMENTS);
return result_column[0];
}
static Field extractValueFromNode(ASTPtr & node, const IDataType & type, const Context & context)
{
if (ASTLiteral * lit = typeid_cast<ASTLiteral *>(node.get()))
return convertToType(lit->value, type);
else if (typeid_cast<ASTFunction *>(node.get()))
return convertToType(evaluateConstantExpression(node, context), type);
else
throw Exception("Incorrect element of set. Must be literal or constant expression.", ErrorCodes::INCORRECT_ELEMENT_OF_SET);
}
void Set::createFromAST(DataTypes & types, ASTPtr node, const Context & context, bool create_ordered_set)
{
data_types = types; data_types = types;
/// Засунем множество в блок. /// Засунем множество в блок.
@ -359,10 +422,7 @@ void Set::createFromAST(DataTypes & types, ASTPtr node, bool create_ordered_set)
{ {
if (data_types.size() == 1) if (data_types.size() == 1)
{ {
if (ASTLiteral * lit = typeid_cast<ASTLiteral *>(&**it)) block.getByPosition(0).column->insert(extractValueFromNode(*it, *data_types[0], context));
block.getByPosition(0).column->insert(convertToType(lit->value, *data_types[0]));
else
throw Exception("Incorrect element of set. Must be literal.", ErrorCodes::INCORRECT_ELEMENT_OF_SET);
} }
else if (ASTFunction * func = typeid_cast<ASTFunction *>(&**it)) else if (ASTFunction * func = typeid_cast<ASTFunction *>(&**it))
{ {
@ -375,16 +435,11 @@ void Set::createFromAST(DataTypes & types, ASTPtr node, bool create_ordered_set)
for (size_t j = 0; j < tuple_size; ++j) for (size_t j = 0; j < tuple_size; ++j)
{ {
if (ASTLiteral * lit = typeid_cast<ASTLiteral *>(&*func->arguments->children[j])) block.getByPosition(j).column->insert(extractValueFromNode(func->arguments->children[j], *data_types[j], context));
block.getByPosition(j).column->insert(convertToType(lit->value, *data_types[j]));
else
throw Exception("Incorrect element of tuple in set. Must be literal.", ErrorCodes::INCORRECT_ELEMENT_OF_SET);
} }
} }
else else
throw Exception("Incorrect element of set", ErrorCodes::INCORRECT_ELEMENT_OF_SET); throw Exception("Incorrect element of set", ErrorCodes::INCORRECT_ELEMENT_OF_SET);
/// NOTE: Потом можно реализовать возможность задавать константные выражения в множествах.
} }
if (create_ordered_set) if (create_ordered_set)

View File

@ -164,6 +164,7 @@ ASTPtr ASTSelectQuery::clone() const
CLONE(order_expression_list) CLONE(order_expression_list)
CLONE(limit_offset) CLONE(limit_offset)
CLONE(limit_length) CLONE(limit_length)
CLONE(settings)
CLONE(format) CLONE(format)
CLONE(next_union_all) CLONE(next_union_all)

View File

@ -112,6 +112,7 @@ bool ParserLeftAssociativeBinaryOperatorList::parseImpl(Pos & pos, Pos end, ASTP
{ {
bool first = true; bool first = true;
ParserWhiteSpaceOrComments ws; ParserWhiteSpaceOrComments ws;
Pos begin = pos;
while (1) while (1)
{ {
@ -129,7 +130,6 @@ bool ParserLeftAssociativeBinaryOperatorList::parseImpl(Pos & pos, Pos end, ASTP
ws.ignore(pos, end); ws.ignore(pos, end);
/// пробуем найти какой-нибудь из допустимых операторов /// пробуем найти какой-нибудь из допустимых операторов
Pos begin = pos;
const char ** it; const char ** it;
for (it = operators; *it; it += 2) for (it = operators; *it; it += 2)

View File

@ -5,6 +5,7 @@
#include <DB/Parsers/ExpressionElementParsers.h> #include <DB/Parsers/ExpressionElementParsers.h>
#include <DB/Parsers/ExpressionListParsers.h> #include <DB/Parsers/ExpressionListParsers.h>
#include <DB/Parsers/ParserJoin.h> #include <DB/Parsers/ParserJoin.h>
#include <DB/Parsers/ParserSetQuery.h>
#include <DB/Parsers/ParserSelectQuery.h> #include <DB/Parsers/ParserSelectQuery.h>
@ -37,6 +38,7 @@ bool ParserSelectQuery::parseImpl(Pos & pos, Pos end, ASTPtr & node, Pos & max_p
ParserString s_having("HAVING", true, true); ParserString s_having("HAVING", true, true);
ParserString s_order("ORDER", true, true); ParserString s_order("ORDER", true, true);
ParserString s_limit("LIMIT", true, true); ParserString s_limit("LIMIT", true, true);
ParserString s_settings("SETTINGS", true, true);
ParserString s_format("FORMAT", true, true); ParserString s_format("FORMAT", true, true);
ParserString s_union("UNION", true, true); ParserString s_union("UNION", true, true);
ParserString s_all("ALL", true, true); ParserString s_all("ALL", true, true);
@ -281,6 +283,19 @@ bool ParserSelectQuery::parseImpl(Pos & pos, Pos end, ASTPtr & node, Pos & max_p
} }
} }
/// SETTINGS key1 = value1, key2 = value2, ...
if (s_settings.ignore(pos, end, max_parsed_pos, expected))
{
ws.ignore(pos, end);
ParserSetQuery parser_settings(true);
if (!parser_settings.parse(pos, end, select_query->settings, max_parsed_pos, expected))
return false;
ws.ignore(pos, end);
}
/// FORMAT format_name /// FORMAT format_name
if (s_format.ignore(pos, end, max_parsed_pos, expected)) if (s_format.ignore(pos, end, max_parsed_pos, expected))
{ {
@ -339,6 +354,8 @@ bool ParserSelectQuery::parseImpl(Pos & pos, Pos end, ASTPtr & node, Pos & max_p
select_query->children.push_back(select_query->limit_offset); select_query->children.push_back(select_query->limit_offset);
if (select_query->limit_length) if (select_query->limit_length)
select_query->children.push_back(select_query->limit_length); select_query->children.push_back(select_query->limit_length);
if (select_query->settings)
select_query->children.push_back(select_query->settings);
if (select_query->format) if (select_query->format)
select_query->children.push_back(select_query->format); select_query->children.push_back(select_query->format);
if (select_query->next_union_all) if (select_query->next_union_all)

View File

@ -50,18 +50,24 @@ bool ParserSetQuery::parseImpl(Pos & pos, Pos end, ASTPtr & node, Pos & max_pars
Pos begin = pos; Pos begin = pos;
ParserWhiteSpaceOrComments ws; ParserWhiteSpaceOrComments ws;
ParserString s_set("SET", true, true);
ParserString s_global("GLOBAL", true, true);
ParserString s_comma(","); ParserString s_comma(",");
ws.ignore(pos, end); bool global = false;
if (!s_set.ignore(pos, end, max_parsed_pos, expected)) if (!parse_only_internals)
return false; {
ParserString s_set("SET", true, true);
ParserString s_global("GLOBAL", true, true);
ws.ignore(pos, end); ws.ignore(pos, end);
bool global = s_global.ignore(pos, end, max_parsed_pos, expected); if (!s_set.ignore(pos, end, max_parsed_pos, expected))
return false;
ws.ignore(pos, end);
global = s_global.ignore(pos, end, max_parsed_pos, expected);
}
ASTSetQuery::Changes changes; ASTSetQuery::Changes changes;

View File

@ -211,6 +211,20 @@ void formatAST(const ASTSelectQuery & ast, std::ostream & s, size_t indent, bo
formatAST(*ast.limit_length, s, indent, hilite, one_line); formatAST(*ast.limit_length, s, indent, hilite, one_line);
} }
if (ast.settings)
{
s << (hilite ? hilite_keyword : "") << nl_or_ws << indent_str << "SETTINGS " << (hilite ? hilite_none : "");
const ASTSetQuery & ast_set = typeid_cast<const ASTSetQuery &>(*ast.settings);
for (ASTSetQuery::Changes::const_iterator it = ast_set.changes.begin(); it != ast_set.changes.end(); ++it)
{
if (it != ast_set.changes.begin())
s << ", ";
s << it->name << " = " << apply_visitor(FieldVisitorToString(), it->value);
}
}
if (ast.format) if (ast.format)
{ {
s << (hilite ? hilite_keyword : "") << nl_or_ws << indent_str << "FORMAT " << (hilite ? hilite_none : ""); s << (hilite ? hilite_keyword : "") << nl_or_ws << indent_str << "FORMAT " << (hilite ? hilite_none : "");

View File

@ -64,8 +64,15 @@ static std::string getSyntaxErrorMessage(
{ {
message << ":\n\n"; message << ":\n\n";
message.write(begin, max_parsed_pos - begin); message.write(begin, max_parsed_pos - begin);
message << "\033[41;1m" << *max_parsed_pos << "\033[0m"; /// Ярко-красный фон.
message.write(max_parsed_pos + 1, end - max_parsed_pos - 1); size_t bytes_to_hilite = 1;
while (max_parsed_pos + bytes_to_hilite < end
&& static_cast<unsigned char>(max_parsed_pos[bytes_to_hilite]) >= 0x80 /// UTF-8
&& static_cast<unsigned char>(max_parsed_pos[bytes_to_hilite]) <= 0xBF)
++bytes_to_hilite;
message << "\033[41;1m" << std::string(max_parsed_pos, bytes_to_hilite) << "\033[0m"; /// Ярко-красный фон.
message.write(max_parsed_pos + bytes_to_hilite, end - max_parsed_pos - bytes_to_hilite);
message << "\n\n"; message << "\n\n";
if (expected && *expected && *expected != '.') if (expected && *expected && *expected != '.')

View File

@ -220,10 +220,12 @@ bool MergeTreeDataMerger::selectPartsToMerge(MergeTreeData::DataPartsVector & pa
{ {
disk_space_warning_time = now; disk_space_warning_time = now;
LOG_WARNING(log, "Won't merge parts from " << first_part->name << " to " << last_part->name LOG_WARNING(log, "Won't merge parts from " << first_part->name << " to " << last_part->name
<< " because not enough free space: " << available_disk_space << " free and unreserved " << " because not enough free space: "
<< "(" << DiskSpaceMonitor::getReservedSpace() << " reserved in " << formatReadableSizeWithBinarySuffix(available_disk_space) << " free and unreserved "
<< "(" << formatReadableSizeWithBinarySuffix(DiskSpaceMonitor::getReservedSpace()) << " reserved in "
<< DiskSpaceMonitor::getReservationCount() << " chunks), " << DiskSpaceMonitor::getReservationCount() << " chunks), "
<< cur_sum << " required now (+" << static_cast<int>((DISK_USAGE_COEFFICIENT_TO_SELECT - 1.0) * 100) << formatReadableSizeWithBinarySuffix(cur_sum)
<< " required now (+" << static_cast<int>((DISK_USAGE_COEFFICIENT_TO_SELECT - 1.0) * 100)
<< "% on overhead); suppressing similar warnings for the next hour"); << "% on overhead); suppressing similar warnings for the next hour");
} }
break; break;
@ -461,9 +463,8 @@ size_t MergeTreeDataMerger::estimateDiskSpaceForMerge(const MergeTreeData::DataP
{ {
size_t res = 0; size_t res = 0;
for (const MergeTreeData::DataPartPtr & part : parts) for (const MergeTreeData::DataPartPtr & part : parts)
{
res += part->size_in_bytes; res += part->size_in_bytes;
}
return static_cast<size_t>(res * DISK_USAGE_COEFFICIENT_TO_RESERVE); return static_cast<size_t>(res * DISK_USAGE_COEFFICIENT_TO_RESERVE);
} }

View File

@ -13,19 +13,29 @@ namespace DB
{ {
StoragePtr StorageBuffer::create(const std::string & name_, NamesAndTypesListPtr columns_, Context & context_, StoragePtr StorageBuffer::create(const std::string & name_, NamesAndTypesListPtr columns_,
const NamesAndTypesList & materialized_columns_,
const NamesAndTypesList & alias_columns_,
const ColumnDefaults & column_defaults_,
Context & context_,
size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_, size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_,
const String & destination_database_, const String & destination_table_) const String & destination_database_, const String & destination_table_)
{ {
return (new StorageBuffer{ return (new StorageBuffer{
name_, columns_, context_, num_shards_, min_thresholds_, max_thresholds_, destination_database_, destination_table_})->thisPtr(); name_, columns_, materialized_columns_, alias_columns_, column_defaults_,
context_, num_shards_, min_thresholds_, max_thresholds_, destination_database_, destination_table_})->thisPtr();
} }
StorageBuffer::StorageBuffer(const std::string & name_, NamesAndTypesListPtr columns_, Context & context_, StorageBuffer::StorageBuffer(const std::string & name_, NamesAndTypesListPtr columns_,
const NamesAndTypesList & materialized_columns_,
const NamesAndTypesList & alias_columns_,
const ColumnDefaults & column_defaults_,
Context & context_,
size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_, size_t num_shards_, const Thresholds & min_thresholds_, const Thresholds & max_thresholds_,
const String & destination_database_, const String & destination_table_) const String & destination_database_, const String & destination_table_)
: name(name_), columns(columns_), context(context_), : IStorage{materialized_columns_, alias_columns_, column_defaults_},
name(name_), columns(columns_), context(context_),
num_shards(num_shards_), buffers(num_shards_), num_shards(num_shards_), buffers(num_shards_),
min_thresholds(min_thresholds_), max_thresholds(max_thresholds_), min_thresholds(min_thresholds_), max_thresholds(max_thresholds_),
destination_database(destination_database_), destination_table(destination_table_), destination_database(destination_database_), destination_table(destination_table_),
@ -43,7 +53,7 @@ public:
BufferBlockInputStream(const Names & column_names_, StorageBuffer::Buffer & buffer_) BufferBlockInputStream(const Names & column_names_, StorageBuffer::Buffer & buffer_)
: column_names(column_names_.begin(), column_names_.end()), buffer(buffer_) {} : column_names(column_names_.begin(), column_names_.end()), buffer(buffer_) {}
String getName() const { return "BufferBlockInputStream"; } String getName() const { return "Buffer"; }
String getID() const String getID() const
{ {

View File

@ -299,7 +299,33 @@ StoragePtr StorageFactory::get(
if (args.size() != 3 && args.size() != 4) if (args.size() != 3 && args.size() != 4)
throw Exception(params_error_message, ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH); throw Exception(params_error_message, ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
String cluster_name = typeid_cast<ASTIdentifier &>(*args[0]).name; /** Имя кластера - это имя тега в xml-конфигурации.
* Обычно оно парсится как идентификатор. То есть, оно может содержать подчёркивания, но не может содержать дефисы,
* при условии, что идентификатор не находится в обратных кавычках.
* Но в xml в качестве имени тега более привычно использовать дефисы.
* Такое имя будет парситься как выражение с оператором минус - совсем не то, что нужно.
* Поэтому, рассмотрим такой случай отдельно.
*/
String cluster_name;
if (const ASTIdentifier * ast_id = typeid_cast<const ASTIdentifier *>(args[0].get()))
{
cluster_name = ast_id->name;
}
else if (const ASTLiteral * ast_lit = typeid_cast<const ASTLiteral *>(args[0].get()))
{
cluster_name = ast_lit->value.safeGet<String>();
}
else if (const ASTFunction * ast_func = typeid_cast<const ASTFunction *>(args[0].get()))
{
if (!ast_func->range.first || !ast_func->range.second)
throw Exception("Illegal expression instead of cluster name.", ErrorCodes::BAD_ARGUMENTS);
cluster_name = String(ast_func->range.first, ast_func->range.second);
}
else
throw Exception("Illegal expression instead of cluster name.", ErrorCodes::BAD_ARGUMENTS);
String remote_database = reinterpretAsIdentifier(args[1], local_context).name; String remote_database = reinterpretAsIdentifier(args[1], local_context).name;
String remote_table = typeid_cast<ASTIdentifier &>(*args[2]).name; String remote_table = typeid_cast<ASTIdentifier &>(*args[2]).name;
@ -347,7 +373,9 @@ StoragePtr StorageFactory::get(
size_t max_bytes = apply_visitor(FieldVisitorConvertToNumber<size_t>(), typeid_cast<ASTLiteral &>(*args[8]).value); size_t max_bytes = apply_visitor(FieldVisitorConvertToNumber<size_t>(), typeid_cast<ASTLiteral &>(*args[8]).value);
return StorageBuffer::create( return StorageBuffer::create(
table_name, columns, context, table_name, columns,
materialized_columns, alias_columns, column_defaults,
context,
num_buckets, {min_time, min_rows, min_bytes}, {max_time, max_rows, max_bytes}, num_buckets, {min_time, min_rows, min_bytes}, {max_time, max_rows, max_bytes},
destination_database, destination_table); destination_database, destination_table);
} }

View File

@ -46,7 +46,7 @@ public:
: block_size(block_size_), column_names(column_names_), storage(storage_), : block_size(block_size_), column_names(column_names_), storage(storage_),
mark_number(mark_number_), rows_limit(rows_limit_), current_mark(mark_number_), max_read_buffer_size(max_read_buffer_size_) {} mark_number(mark_number_), rows_limit(rows_limit_), current_mark(mark_number_), max_read_buffer_size(max_read_buffer_size_) {}
String getName() const { return "LogBlockInputStream"; } String getName() const { return "Log"; }
String getID() const String getID() const
{ {

View File

@ -20,7 +20,7 @@ public:
MemoryBlockInputStream(const Names & column_names_, BlocksList::iterator begin_, BlocksList::iterator end_) MemoryBlockInputStream(const Names & column_names_, BlocksList::iterator begin_, BlocksList::iterator end_)
: column_names(column_names_), begin(begin_), end(end_), it(begin) {} : column_names(column_names_), begin(begin_), end(end_), it(begin) {}
String getName() const { return "MemoryBlockInputStream"; } String getName() const { return "Memory"; }
String getID() const String getID() const
{ {

View File

@ -43,6 +43,25 @@ StorageMergeTree::StorageMergeTree(
data.loadDataParts(false); data.loadDataParts(false);
data.clearOldParts(); data.clearOldParts();
increment.set(data.getMaxDataPartIndex()); increment.set(data.getMaxDataPartIndex());
/** Если остался старый (не использующийся сейчас) файл increment.txt, то удалим его.
* Это нужно сделать, чтобы избежать ситуации, когда из-за копирования данных
* от сервера с новой версией (но с оставшимся некорректным и неиспользуемым increment.txt)
* на сервер со старой версией (где increment.txt используется),
* будет скопирован и использован некорректный increment.txt.
*
* Это - защита от очень редкого гипотетического случая.
* Он может достигаться в БК, где довольно медленно обновляют ПО,
* но зато часто делают копирование данных rsync-ом.
*/
{
Poco::File obsolete_increment_txt(full_path + "increment.txt");
if (obsolete_increment_txt.exists())
{
LOG_INFO(log, "Removing obsolete file " << full_path << "increment.txt");
obsolete_increment_txt.remove();
}
}
} }
StoragePtr StorageMergeTree::create( StoragePtr StorageMergeTree::create(
@ -210,7 +229,7 @@ bool StorageMergeTree::merge(size_t aio_threshold, bool aggressive, BackgroundPr
return false; return false;
} }
merging_tagger = new CurrentlyMergingPartsTagger(parts, merger.estimateDiskSpaceForMerge(parts), *this); merging_tagger = new CurrentlyMergingPartsTagger(parts, MergeTreeDataMerger::estimateDiskSpaceForMerge(parts), *this);
/// Если собираемся сливать большие куски, увеличим счетчик потоков, сливающих большие куски. /// Если собираемся сливать большие куски, увеличим счетчик потоков, сливающих большие куски.
if (pool_context) if (pool_context)

View File

@ -14,6 +14,7 @@
#include <DB/Parsers/ASTInsertQuery.h> #include <DB/Parsers/ASTInsertQuery.h>
#include <DB/DataStreams/AddingConstColumnBlockInputStream.h> #include <DB/DataStreams/AddingConstColumnBlockInputStream.h>
#include <DB/Common/Macros.h> #include <DB/Common/Macros.h>
#include <DB/Common/formatReadable.h>
#include <Poco/DirectoryIterator.h> #include <Poco/DirectoryIterator.h>
#include <time.h> #include <time.h>
@ -860,12 +861,15 @@ bool StorageReplicatedMergeTree::executeLogEntry(const LogEntry & entry, Backgro
} }
} }
size_t sum_parts_size_in_bytes = MergeTreeDataMerger::estimateDiskSpaceForMerge(parts);
DiskSpaceMonitor::ReservationPtr reserved_space = DiskSpaceMonitor::reserve(full_path, sum_parts_size_in_bytes); /// Может бросить исключение.
auto table_lock = lockStructure(false); auto table_lock = lockStructure(false);
const auto & merge_entry = context.getMergeList().insert(database_name, table_name, entry.new_part_name); const auto & merge_entry = context.getMergeList().insert(database_name, table_name, entry.new_part_name);
MergeTreeData::Transaction transaction; MergeTreeData::Transaction transaction;
size_t aio_threshold = context.getSettings().min_bytes_to_use_direct_io; size_t aio_threshold = context.getSettings().min_bytes_to_use_direct_io;
MergeTreeData::DataPartPtr part = merger.mergeParts(parts, entry.new_part_name, *merge_entry, aio_threshold, &transaction); MergeTreeData::DataPartPtr part = merger.mergeParts(parts, entry.new_part_name, *merge_entry, aio_threshold, &transaction, reserved_space);
zkutil::Ops ops; zkutil::Ops ops;
checkPartAndAddToZooKeeper(part, ops); checkPartAndAddToZooKeeper(part, ops);
@ -1360,8 +1364,10 @@ void StorageReplicatedMergeTree::mergeSelectingThread()
String merged_name; String merged_name;
if ( !merger.selectPartsToMerge(parts, merged_name, MergeTreeDataMerger::NO_LIMIT, false, false, only_small, can_merge) size_t disk_space = DiskSpaceMonitor::getUnreservedFreeSpace(full_path);
&& !merger.selectPartsToMerge(parts, merged_name, MergeTreeDataMerger::NO_LIMIT, true, false, only_small, can_merge))
if ( !merger.selectPartsToMerge(parts, merged_name, disk_space, false, false, only_small, can_merge)
&& !merger.selectPartsToMerge(parts, merged_name, disk_space, true, false, only_small, can_merge))
{ {
break; break;
} }

View File

@ -19,7 +19,7 @@ public:
NumbersBlockInputStream(size_t block_size_, size_t offset_, size_t step_) NumbersBlockInputStream(size_t block_size_, size_t offset_, size_t step_)
: block_size(block_size_), next(offset_), step(step_) {} : block_size(block_size_), next(offset_), step(step_) {}
String getName() const { return "NumbersBlockInputStream"; } String getName() const { return "Numbers"; }
String getID() const { return "Numbers"; } String getID() const { return "Numbers"; }
protected: protected:

View File

@ -42,7 +42,7 @@ public:
TinyLogBlockInputStream(size_t block_size_, const Names & column_names_, StorageTinyLog & storage_, size_t max_read_buffer_size_) TinyLogBlockInputStream(size_t block_size_, const Names & column_names_, StorageTinyLog & storage_, size_t max_read_buffer_size_)
: block_size(block_size_), column_names(column_names_), storage(storage_), max_read_buffer_size(max_read_buffer_size_) {} : block_size(block_size_), column_names(column_names_), storage(storage_), max_read_buffer_size(max_read_buffer_size_) {}
String getName() const { return "TinyLogBlockInputStream"; } String getName() const { return "TinyLog"; }
String getID() const; String getID() const;

View File

@ -1,6 +1,6 @@
DROP TABLE IF EXISTS test.join; DROP TABLE IF EXISTS test.join;
CREATE TABLE test.join (k UInt8, s String) ENGINE = Join(ANY, LEFT, k); CREATE TABLE test.join (k UInt64, s String) ENGINE = Join(ANY, LEFT, k);
USE test; USE test;

View File

@ -1,6 +1,6 @@
DROP TABLE IF EXISTS test.join; DROP TABLE IF EXISTS test.join;
CREATE TABLE test.join (s String, x Array(UInt8), k UInt8) ENGINE = Join(ANY, LEFT, k); CREATE TABLE test.join (s String, x Array(UInt8), k UInt64) ENGINE = Join(ANY, LEFT, k);
USE test; USE test;

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