mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-23 08:02:02 +00:00
Added requested changes. [#METR-22071]
This commit is contained in:
parent
37ed6a3038
commit
946a037f71
@ -110,9 +110,9 @@ public:
|
||||
nested_func->serialize(place, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena * arena) const override
|
||||
{
|
||||
nested_func->deserialize(place, buf);
|
||||
nested_func->deserialize(place, buf, arena);
|
||||
}
|
||||
|
||||
void insertResultInto(ConstAggregateDataPtr place, IColumn & to) const override
|
||||
|
@ -60,7 +60,7 @@ public:
|
||||
writeVarUInt(this->data(place).count, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
readBinary(this->data(place).sum, buf);
|
||||
readVarUInt(this->data(place).count, buf);
|
||||
|
@ -46,7 +46,7 @@ public:
|
||||
writeVarUInt(data(place).count, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
readVarUInt(data(place).count, buf);
|
||||
}
|
||||
|
@ -69,7 +69,7 @@ public:
|
||||
buf.write(reinterpret_cast<const char *>(&value[0]), size * sizeof(value[0]));
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
size_t size = 0;
|
||||
readVarUInt(size, buf);
|
||||
@ -148,7 +148,7 @@ public:
|
||||
type->serializeBinary(value[i], buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
size_t size = 0;
|
||||
readVarUInt(size, buf);
|
||||
|
@ -68,14 +68,14 @@ public:
|
||||
|
||||
void serialize(ConstAggregateDataPtr place, WriteBuffer & buf) const override
|
||||
{
|
||||
const typename State::Set & set = this->data(place).value;
|
||||
auto & set = this->data(place).value;
|
||||
size_t size = set.size();
|
||||
writeVarUInt(size, buf);
|
||||
for (auto it = set.begin(); it != set.end(); ++it)
|
||||
writeIntBinary(*it, buf);
|
||||
for (auto & elem : set)
|
||||
writeIntBinary(elem, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).value.read(buf);
|
||||
}
|
||||
@ -108,10 +108,13 @@ struct AggreagteFunctionGroupUniqArrayGenericData
|
||||
Set value;
|
||||
};
|
||||
|
||||
template <bool is_plain_column=false>
|
||||
/** Template parameter with true value should be used for columns that store their elements in memory continuously.
|
||||
* For such columns groupUniqArray() can be implemented more efficently (especially for small numeric arrays).
|
||||
*/
|
||||
template <bool is_plain_column = false>
|
||||
class AggreagteFunctionGroupUniqArrayGeneric : public IUnaryAggregateFunction<AggreagteFunctionGroupUniqArrayGenericData, AggreagteFunctionGroupUniqArrayGeneric<is_plain_column>>
|
||||
{
|
||||
mutable DataTypePtr input_data_type;
|
||||
DataTypePtr input_data_type;
|
||||
|
||||
using State = AggreagteFunctionGroupUniqArrayGenericData;
|
||||
|
||||
@ -138,24 +141,24 @@ public:
|
||||
auto & set = this->data(place).value;
|
||||
writeVarUInt(set.size(), buf);
|
||||
|
||||
for (auto & elem: set)
|
||||
for (const auto & elem : set)
|
||||
{
|
||||
writeStringBinary(elem, buf);
|
||||
}
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena * arena) const override
|
||||
{
|
||||
State::Set & set = this->data(place).value;
|
||||
auto & set = this->data(place).value;
|
||||
size_t size;
|
||||
readVarUInt(size, buf);
|
||||
//TODO: set.reserve(size);
|
||||
|
||||
std::string str_buf;
|
||||
arena = new Arena(size * 10);
|
||||
|
||||
for (size_t i = 0; i < size; i++)
|
||||
{
|
||||
readStringBinary(str_buf, buf);
|
||||
set.insert(StringRef(str_buf));
|
||||
set.insert(readStringBinaryInto(*arena, buf));
|
||||
}
|
||||
}
|
||||
|
||||
@ -171,12 +174,12 @@ public:
|
||||
|
||||
if (!is_plain_column)
|
||||
{
|
||||
if (!likely(inserted))
|
||||
if (!inserted)
|
||||
arena->rollback(str_serialized.size);
|
||||
}
|
||||
else
|
||||
{
|
||||
if (likely(inserted))
|
||||
if (inserted)
|
||||
it->data = arena->insert(str_serialized.data, str_serialized.size);
|
||||
}
|
||||
}
|
||||
|
@ -93,9 +93,9 @@ public:
|
||||
nested_func->serialize(place, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena * arena) const override
|
||||
{
|
||||
nested_func->deserialize(place, buf);
|
||||
nested_func->deserialize(place, buf, arena);
|
||||
}
|
||||
|
||||
void insertResultInto(ConstAggregateDataPtr place, IColumn & to) const override
|
||||
|
@ -94,9 +94,9 @@ public:
|
||||
nested_func->serialize(place, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena * arena) const override
|
||||
{
|
||||
nested_func->deserialize(place, buf);
|
||||
nested_func->deserialize(place, buf, arena);
|
||||
}
|
||||
|
||||
void insertResultInto(ConstAggregateDataPtr place, IColumn & to) const override
|
||||
|
@ -83,7 +83,7 @@ public:
|
||||
this->data(place).sample.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).sample.read(buf);
|
||||
}
|
||||
@ -160,7 +160,7 @@ public:
|
||||
this->data(place).sample.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).sample.read(buf);
|
||||
}
|
||||
|
@ -90,7 +90,7 @@ public:
|
||||
this->data(place).sample.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).sample.read(buf);
|
||||
}
|
||||
@ -174,7 +174,7 @@ public:
|
||||
this->data(place).sample.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).sample.read(buf);
|
||||
}
|
||||
|
@ -87,7 +87,7 @@ public:
|
||||
buf.write(reinterpret_cast<const char *>(&array[0]), size * sizeof(array[0]));
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
auto & array = this->data(place).array;
|
||||
|
||||
@ -169,7 +169,7 @@ public:
|
||||
buf.write(reinterpret_cast<const char *>(&array[0]), size * sizeof(array[0]));
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
auto & array = this->data(place).array;
|
||||
|
||||
|
@ -95,7 +95,7 @@ public:
|
||||
this->data(place).map.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
typename AggregateFunctionQuantileExactWeightedData<ValueType>::Map::Reader reader(buf);
|
||||
|
||||
@ -210,7 +210,7 @@ public:
|
||||
this->data(place).map.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
typename AggregateFunctionQuantileExactWeightedData<ValueType>::Map::Reader reader(buf);
|
||||
|
||||
|
@ -397,7 +397,7 @@ public:
|
||||
this->data(const_cast<AggregateDataPtr>(place)).digest.write(params, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).digest.read(params, buf);
|
||||
}
|
||||
@ -466,7 +466,7 @@ public:
|
||||
this->data(const_cast<AggregateDataPtr>(place)).digest.write(params, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).digest.read(params, buf);
|
||||
}
|
||||
@ -528,7 +528,7 @@ public:
|
||||
this->data(const_cast<AggregateDataPtr>(place)).digest.write(params, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).digest.read(params, buf);
|
||||
}
|
||||
@ -610,7 +610,7 @@ public:
|
||||
this->data(const_cast<AggregateDataPtr>(place)).digest.write(params, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).digest.read(params, buf);
|
||||
}
|
||||
|
@ -830,7 +830,7 @@ public:
|
||||
this->data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).deserialize(buf);
|
||||
}
|
||||
@ -890,7 +890,7 @@ public:
|
||||
this->data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).deserialize(buf);
|
||||
}
|
||||
@ -945,7 +945,7 @@ public:
|
||||
this->data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).deserialize(buf);
|
||||
}
|
||||
@ -1008,7 +1008,7 @@ public:
|
||||
this->data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).deserialize(buf);
|
||||
}
|
||||
|
@ -216,7 +216,7 @@ public:
|
||||
data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
data(place).deserialize(buf);
|
||||
}
|
||||
|
@ -87,9 +87,9 @@ public:
|
||||
nested_func->serialize(place, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena * arena) const override
|
||||
{
|
||||
nested_func->deserialize(place, buf);
|
||||
nested_func->deserialize(place, buf, arena);
|
||||
}
|
||||
|
||||
void insertResultInto(ConstAggregateDataPtr place, IColumn & to) const override
|
||||
|
@ -55,7 +55,7 @@ public:
|
||||
writeBinary(this->data(place).sum, buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
readBinary(this->data(place).sum, buf);
|
||||
}
|
||||
|
@ -355,7 +355,7 @@ public:
|
||||
this->data(place).set.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).set.read(buf);
|
||||
}
|
||||
@ -410,7 +410,7 @@ public:
|
||||
this->data(place).set.write(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).set.read(buf);
|
||||
}
|
||||
|
@ -166,7 +166,7 @@ public:
|
||||
this->data(place).write(buf, threshold);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).read(buf, threshold);
|
||||
}
|
||||
@ -239,7 +239,7 @@ public:
|
||||
this->data(place).write(buf, threshold);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).read(buf, threshold);
|
||||
}
|
||||
|
@ -59,7 +59,7 @@ public:
|
||||
this->data(place).value.write(buf, *type_val.get());
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).result.read(buf, *type_res.get());
|
||||
this->data(place).value.read(buf, *type_val.get());
|
||||
|
@ -664,7 +664,7 @@ public:
|
||||
this->data(place).write(buf, *type.get());
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).read(buf, *type.get());
|
||||
}
|
||||
|
@ -144,7 +144,7 @@ public:
|
||||
this->data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).deserialize(buf);
|
||||
}
|
||||
@ -412,7 +412,7 @@ public:
|
||||
this->data(place).serialize(buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
this->data(place).deserialize(buf);
|
||||
}
|
||||
|
@ -77,19 +77,23 @@ public:
|
||||
/// Как должна быть выровнена структура с данными. NOTE: Сейчас не используется (структуры с состоянием агрегации кладутся без выравнивания).
|
||||
virtual size_t alignOfData() const = 0;
|
||||
|
||||
/// Добавить значение. columns - столбцы, содержащие аргументы, row_num - номер строки в столбцах.
|
||||
/** Adds a value into aggregation data on which place points to.
|
||||
* columns points to columns containing arguments of aggregation function.
|
||||
* row_num is number of row which should be added.
|
||||
* Additional parameter arena should be used instead of standard memory allocator if the addition requires memory allocation.
|
||||
*/
|
||||
virtual void add(AggregateDataPtr place, const IColumn ** columns, size_t row_num, Arena * arena) const = 0;
|
||||
|
||||
/// Объединить состояние с другим состоянием.
|
||||
/// Merges state (on which place points to) with other state of current aggregation function.
|
||||
virtual void merge(AggregateDataPtr place, ConstAggregateDataPtr rhs) const = 0;
|
||||
|
||||
/// Сериализовать состояние (например, для передачи по сети).
|
||||
/// Serializes state (to transmit it over the network, for example).
|
||||
virtual void serialize(ConstAggregateDataPtr place, WriteBuffer & buf) const = 0;
|
||||
|
||||
/// Десериализовать состояние. Вызывается для пустого (только что созданного) состояния.
|
||||
virtual void deserialize(AggregateDataPtr place, ReadBuffer & buf) const = 0;
|
||||
/// Deserializes state. This function is called only for empty (just created) states.
|
||||
virtual void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena * arena) const = 0;
|
||||
|
||||
/// Вставить результат в столбец.
|
||||
/// Inserts results into a column.
|
||||
virtual void insertResultInto(ConstAggregateDataPtr place, IColumn & to) const = 0;
|
||||
|
||||
/** Возвращает true для агрегатных функций типа -State.
|
||||
@ -145,11 +149,6 @@ public:
|
||||
{
|
||||
return __alignof__(Data);
|
||||
}
|
||||
|
||||
// bool needArena() const override
|
||||
// {
|
||||
// return std::is_base_of<IAggregateDataWithArena, T>();
|
||||
// }
|
||||
};
|
||||
|
||||
|
||||
|
@ -178,7 +178,7 @@ public:
|
||||
getData().push_back(arena.alloc(function->sizeOfData()));
|
||||
function->create(getData().back());
|
||||
ReadBufferFromString read_buffer(x.get<const String &>());
|
||||
function->deserialize(getData().back(), read_buffer);
|
||||
function->deserialize(getData().back(), read_buffer, &arena);
|
||||
}
|
||||
|
||||
void insertDefault() override
|
||||
|
@ -70,10 +70,6 @@ bool check(const T x) { return x == 0; }
|
||||
template <typename T>
|
||||
void set(T & x) { x = 0; }
|
||||
|
||||
bool check(const std::string & x);
|
||||
|
||||
void set(std::string & x);
|
||||
|
||||
};
|
||||
|
||||
|
||||
|
@ -14,8 +14,10 @@
|
||||
#include <common/LocalDateTime.h>
|
||||
|
||||
#include <DB/Core/Types.h>
|
||||
#include <DB/Core/StringRef.h>
|
||||
#include <DB/Common/Exception.h>
|
||||
#include <DB/Common/StringUtils.h>
|
||||
#include <DB/Common/Arena.h>
|
||||
|
||||
#include <DB/IO/ReadBuffer.h>
|
||||
#include <DB/IO/VarInt.h>
|
||||
@ -126,6 +128,18 @@ inline void readStringBinary(std::string & s, ReadBuffer & buf, size_t MAX_STRIN
|
||||
}
|
||||
|
||||
|
||||
inline StringRef readStringBinaryInto(Arena & arena, ReadBuffer & buf)
|
||||
{
|
||||
size_t size = 0;
|
||||
readVarUInt(size, buf);
|
||||
|
||||
char * data = arena.alloc(size);
|
||||
buf.readStrict(data, size);
|
||||
|
||||
return StringRef(data, size);
|
||||
}
|
||||
|
||||
|
||||
template <typename T>
|
||||
void readVectorBinary(std::vector<T> & v, ReadBuffer & buf, size_t MAX_VECTOR_SIZE = DEFAULT_MAX_STRING_SIZE)
|
||||
{
|
||||
|
@ -44,7 +44,7 @@ namespace ErrorCodes
|
||||
* Большинство структур данных существует в двух вариантах: обычном и двухуровневом (TwoLevel).
|
||||
* Двухуровневая хэш-таблица работает чуть медленнее при маленьком количестве различных ключей,
|
||||
* но при большом количестве различных ключей лучше масштабируется, так как позволяет
|
||||
* распараллелить некоторые операции (слияние, пост-обработку) естественным образом.
|
||||
* распараллелить некоторые операции (слияние, пост-обработку) естественным образом.
|
||||
*
|
||||
* Чтобы обеспечить эффективную работу в большом диапазоне условий,
|
||||
* сначала используются одноуровневые хэш-таблицы,
|
||||
@ -977,7 +977,7 @@ protected:
|
||||
void initialize(const Block & block);
|
||||
|
||||
/** Установить блок - пример результата,
|
||||
* только если он ещё не был установлен.
|
||||
* только если он ещё не был установлен.
|
||||
*/
|
||||
void setSampleBlock(const Block & block);
|
||||
|
||||
@ -986,7 +986,7 @@ protected:
|
||||
|
||||
/** Создать состояния агрегатных функций для одного ключа.
|
||||
*/
|
||||
void createAggregateStates(AggregateDataPtr & aggregate_data, Arena * arena) const;
|
||||
void createAggregateStates(AggregateDataPtr & aggregate_data) const;
|
||||
|
||||
/** Вызвать методы destroy для состояний агрегатных функций.
|
||||
* Используется в обработчике исключений при агрегации, так как RAII в данном случае не применим.
|
||||
|
@ -63,7 +63,7 @@ public:
|
||||
writeBinary(UInt8(0), buf);
|
||||
}
|
||||
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf) const override
|
||||
void deserialize(AggregateDataPtr place, ReadBuffer & buf, Arena *) const override
|
||||
{
|
||||
UInt8 tmp;
|
||||
readBinary(tmp, buf);
|
||||
|
@ -1,7 +1,6 @@
|
||||
#include <DB/AggregateFunctions/AggregateFunctionFactory.h>
|
||||
#include <DB/AggregateFunctions/AggregateFunctionGroupUniqArray.h>
|
||||
#include <DB/AggregateFunctions/Helpers.h>
|
||||
#include <common/logger_useful.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
@ -11,12 +10,12 @@ namespace
|
||||
|
||||
static IAggregateFunction * createWithExtraTypes(const IDataType & argument_type)
|
||||
{
|
||||
if (typeid_cast<const DataTypeDateTime *>(&argument_type)) return new AggregateFunctionGroupUniqArray<UInt32>;
|
||||
else if (typeid_cast<const DataTypeDate *>(&argument_type)) return new AggregateFunctionGroupUniqArray<UInt16>;
|
||||
if (typeid_cast<const DataTypeDateTime *>(&argument_type)) return new AggregateFunctionGroupUniqArray<DataTypeDateTime::FieldType>;
|
||||
else if (typeid_cast<const DataTypeDate *>(&argument_type)) return new AggregateFunctionGroupUniqArray<DataTypeDate::FieldType>;
|
||||
else
|
||||
{
|
||||
/// Check that we can use plain version of AggreagteFunctionGroupUniqArrayGeneric
|
||||
if (typeid_cast<const DataTypeString*>(&argument_type))
|
||||
if (typeid_cast<const DataTypeString*>(&argument_type) || typeid_cast<const DataTypeFixedString*>(&argument_type))
|
||||
return new AggreagteFunctionGroupUniqArrayGeneric<true>;
|
||||
|
||||
auto * array_type = typeid_cast<const DataTypeArray *>(&argument_type);
|
||||
|
@ -62,13 +62,14 @@ void DataTypeAggregateFunction::deserializeBinary(IColumn & column, ReadBuffer &
|
||||
{
|
||||
ColumnAggregateFunction & column_concrete = static_cast<ColumnAggregateFunction &>(column);
|
||||
|
||||
Arena & arena = column_concrete.createOrGetArena();
|
||||
size_t size_of_state = function->sizeOfData();
|
||||
AggregateDataPtr place = column_concrete.createOrGetArena().alloc(size_of_state);
|
||||
AggregateDataPtr place = arena.alloc(size_of_state);
|
||||
|
||||
function->create(place);
|
||||
try
|
||||
{
|
||||
function->deserialize(place, istr);
|
||||
function->deserialize(place, istr, &arena);
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
@ -116,7 +117,7 @@ void DataTypeAggregateFunction::deserializeBinary(IColumn & column, ReadBuffer &
|
||||
|
||||
try
|
||||
{
|
||||
function->deserialize(place, istr);
|
||||
function->deserialize(place, istr, &arena);
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
@ -140,15 +141,16 @@ static void deserializeFromString(const AggregateFunctionPtr & function, IColumn
|
||||
{
|
||||
ColumnAggregateFunction & column_concrete = static_cast<ColumnAggregateFunction &>(column);
|
||||
|
||||
Arena & arena = column_concrete.createOrGetArena();
|
||||
size_t size_of_state = function->sizeOfData();
|
||||
AggregateDataPtr place = column_concrete.createOrGetArena().alloc(size_of_state);
|
||||
AggregateDataPtr place = arena.alloc(size_of_state);
|
||||
|
||||
function->create(place);
|
||||
|
||||
try
|
||||
{
|
||||
ReadBufferFromString istr(s);
|
||||
function->deserialize(place, istr);
|
||||
function->deserialize(place, istr, &arena);
|
||||
}
|
||||
catch (...)
|
||||
{
|
||||
|
@ -432,7 +432,7 @@ AggregatedDataVariants::Type Aggregator::chooseAggregationMethod(const ConstColu
|
||||
}
|
||||
|
||||
|
||||
void Aggregator::createAggregateStates(AggregateDataPtr & aggregate_data, Arena * arena) const
|
||||
void Aggregator::createAggregateStates(AggregateDataPtr & aggregate_data) const
|
||||
{
|
||||
for (size_t j = 0; j < params.aggregates_size; ++j)
|
||||
{
|
||||
@ -558,7 +558,7 @@ void NO_INLINE Aggregator::executeImplCase(
|
||||
method.onNewKey(*it, params.keys_size, i, keys, *aggregates_pool);
|
||||
|
||||
AggregateDataPtr place = aggregates_pool->alloc(total_size_of_aggregate_states);
|
||||
createAggregateStates(place, aggregates_pool);
|
||||
createAggregateStates(place);
|
||||
aggregate_data = place;
|
||||
}
|
||||
else
|
||||
@ -679,7 +679,7 @@ bool Aggregator::executeOnBlock(Block & block, AggregatedDataVariants & result,
|
||||
if ((params.overflow_row || result.type == AggregatedDataVariants::Type::without_key) && !result.without_key)
|
||||
{
|
||||
AggregateDataPtr place = result.aggregates_pool->alloc(total_size_of_aggregate_states);
|
||||
createAggregateStates(place, result.aggregates_pool);
|
||||
createAggregateStates(place);
|
||||
result.without_key = place;
|
||||
}
|
||||
|
||||
@ -1005,7 +1005,6 @@ void NO_INLINE Aggregator::convertToBlockImplFinal(
|
||||
ColumnPlainPtrs & final_aggregate_columns,
|
||||
const Sizes & key_sizes) const
|
||||
{
|
||||
//LOG_DEBUG(log, "convertToBlockImplFinal start");
|
||||
for (const auto & value : data)
|
||||
{
|
||||
method.insertKeyIntoColumns(value, key_columns, params.keys_size, key_sizes);
|
||||
@ -1017,7 +1016,6 @@ void NO_INLINE Aggregator::convertToBlockImplFinal(
|
||||
}
|
||||
|
||||
destroyImpl(method, data); /// NOTE Можно сделать лучше.
|
||||
//LOG_DEBUG(log, "convertToBlockImplFinal exit");
|
||||
}
|
||||
|
||||
template <typename Method, typename Table>
|
||||
@ -1028,7 +1026,6 @@ void NO_INLINE Aggregator::convertToBlockImplNotFinal(
|
||||
AggregateColumnsData & aggregate_columns,
|
||||
const Sizes & key_sizes) const
|
||||
{
|
||||
//LOG_DEBUG(log, "convertToBlockImplFinal start");
|
||||
|
||||
for (auto & value : data)
|
||||
{
|
||||
@ -1040,7 +1037,6 @@ void NO_INLINE Aggregator::convertToBlockImplNotFinal(
|
||||
|
||||
Method::getAggregateData(value.second) = nullptr;
|
||||
}
|
||||
//LOG_DEBUG(log, "convertToBlockImplFinal exit");
|
||||
}
|
||||
|
||||
|
||||
@ -1165,7 +1161,6 @@ BlocksList Aggregator::prepareBlocksAndFillSingleLevel(AggregatedDataVariants &
|
||||
const Sizes & key_sizes,
|
||||
bool final)
|
||||
{
|
||||
//LOG_DEBUG(log, "prepareBlocksAndFillSingleLevel start");
|
||||
#define M(NAME) \
|
||||
else if (data_variants.type == AggregatedDataVariants::Type::NAME) \
|
||||
convertToBlockImpl(*data_variants.NAME, data_variants.NAME->data, \
|
||||
@ -1176,7 +1171,6 @@ BlocksList Aggregator::prepareBlocksAndFillSingleLevel(AggregatedDataVariants &
|
||||
#undef M
|
||||
else
|
||||
throw Exception("Unknown aggregated data variant.", ErrorCodes::UNKNOWN_AGGREGATED_DATA_VARIANT);
|
||||
//LOG_DEBUG(log, "prepareBlocksAndFillSingleLevel exit");
|
||||
};
|
||||
|
||||
BlocksList blocks;
|
||||
@ -1346,8 +1340,6 @@ void NO_INLINE Aggregator::mergeDataImpl(
|
||||
for (size_t i = 0; i < params.aggregates_size; ++i)
|
||||
aggregate_functions[i]->destroy(
|
||||
Method::getAggregateData(it->second) + offsets_of_aggregate_states[i]);
|
||||
|
||||
//LOG_DEBUG(log, "mergeDataImpl");
|
||||
}
|
||||
else
|
||||
{
|
||||
@ -1384,8 +1376,6 @@ void NO_INLINE Aggregator::mergeDataNoMoreKeysImpl(
|
||||
aggregate_functions[i]->destroy(
|
||||
Method::getAggregateData(it->second) + offsets_of_aggregate_states[i]);
|
||||
|
||||
//LOG_DEBUG(log, "mergeDataNoMoreKeysImpl");
|
||||
|
||||
Method::getAggregateData(it->second) = nullptr;
|
||||
}
|
||||
|
||||
@ -1415,8 +1405,6 @@ void NO_INLINE Aggregator::mergeDataOnlyExistingKeysImpl(
|
||||
aggregate_functions[i]->destroy(
|
||||
Method::getAggregateData(it->second) + offsets_of_aggregate_states[i]);
|
||||
|
||||
//LOG_DEBUG(log, "mergeDataOnlyExistingKeysImpl");
|
||||
|
||||
Method::getAggregateData(it->second) = nullptr;
|
||||
}
|
||||
|
||||
@ -1441,8 +1429,6 @@ void NO_INLINE Aggregator::mergeWithoutKeyDataImpl(
|
||||
for (size_t i = 0; i < params.aggregates_size; ++i)
|
||||
aggregate_functions[i]->destroy(current_data + offsets_of_aggregate_states[i]);
|
||||
|
||||
//LOG_DEBUG(log, "mergeWithoutKeyDataImpl");
|
||||
|
||||
current_data = nullptr;
|
||||
}
|
||||
}
|
||||
@ -1458,8 +1444,6 @@ void NO_INLINE Aggregator::mergeSingleLevelDataImpl(
|
||||
/// Все результаты агрегации соединяем с первым.
|
||||
for (size_t i = 1, size = non_empty_data.size(); i < size; ++i)
|
||||
{
|
||||
//LOG_DEBUG(log, "mergeSingleLevelDataImpl for_begin " << i << "/" << size-1);
|
||||
|
||||
if (!checkLimits(res->sizeWithoutOverflowRow(), no_more_keys))
|
||||
break;
|
||||
|
||||
@ -1481,10 +1465,7 @@ void NO_INLINE Aggregator::mergeSingleLevelDataImpl(
|
||||
|
||||
/// current не будет уничтожать состояния агрегатных функций в деструкторе
|
||||
current.aggregator = nullptr;
|
||||
|
||||
//LOG_DEBUG(log, "mergeSingleLevelDataImpl for_end " << i << "/" << size-1);
|
||||
}
|
||||
//LOG_DEBUG(log, "mergeSingleLevelDataImpl exit");
|
||||
}
|
||||
|
||||
|
||||
@ -1817,7 +1798,7 @@ void NO_INLINE Aggregator::mergeStreamsImplCase(
|
||||
method.onNewKey(*it, params.keys_size, i, keys, *aggregates_pool);
|
||||
|
||||
AggregateDataPtr place = aggregates_pool->alloc(total_size_of_aggregate_states);
|
||||
createAggregateStates(place, aggregates_pool);
|
||||
createAggregateStates(place);
|
||||
aggregate_data = place;
|
||||
}
|
||||
else
|
||||
@ -1830,8 +1811,6 @@ void NO_INLINE Aggregator::mergeStreamsImplCase(
|
||||
aggregate_functions[j]->merge(
|
||||
value + offsets_of_aggregate_states[j],
|
||||
(*aggregate_columns[j])[i]);
|
||||
|
||||
LOG_DEBUG(log, "mergeStreamsImplCase");
|
||||
}
|
||||
|
||||
/// Пораньше освобождаем память.
|
||||
@ -1869,7 +1848,7 @@ void NO_INLINE Aggregator::mergeWithoutKeyStreamsImpl(
|
||||
if (!res)
|
||||
{
|
||||
AggregateDataPtr place = result.aggregates_pool->alloc(total_size_of_aggregate_states);
|
||||
createAggregateStates(place, result.aggregates_pool);
|
||||
createAggregateStates(place);
|
||||
res = place;
|
||||
}
|
||||
|
||||
@ -2308,8 +2287,6 @@ void NO_INLINE Aggregator::destroyImpl(
|
||||
Method & method,
|
||||
Table & table) const
|
||||
{
|
||||
//LOG_DEBUG(log, "destroyImpl start");
|
||||
|
||||
for (auto elem : table)
|
||||
{
|
||||
AggregateDataPtr & data = Method::getAggregateData(elem.second);
|
||||
@ -2327,8 +2304,6 @@ void NO_INLINE Aggregator::destroyImpl(
|
||||
|
||||
data = nullptr;
|
||||
}
|
||||
|
||||
//LOG_DEBUG(log, "destroyImpl start");
|
||||
}
|
||||
|
||||
|
||||
|
Loading…
Reference in New Issue
Block a user