Merge remote-tracking branch 'origin/master' into fix-flaky-test-20

This commit is contained in:
Alexey Milovidov 2021-07-20 01:00:25 +03:00
commit 72a76d34f5
184 changed files with 2354 additions and 1131 deletions

View File

@ -18,6 +18,8 @@
#define DATE_LUT_MAX (0xFFFFFFFFU - 86400)
#define DATE_LUT_MAX_DAY_NUM 0xFFFF
/// Max int value of Date32, DATE LUT cache size minus daynum_offset_epoch
#define DATE_LUT_MAX_EXTEND_DAY_NUM (DATE_LUT_SIZE - 16436)
/// A constant to add to time_t so every supported time point becomes non-negative and still has the same remainder of division by 3600.
/// If we treat "remainder of division" operation in the sense of modular arithmetic (not like in C++).
@ -270,6 +272,8 @@ public:
auto getOffsetAtStartOfEpoch() const { return offset_at_start_of_epoch; }
auto getTimeOffsetAtStartOfLUT() const { return offset_at_start_of_lut; }
auto getDayNumOffsetEpoch() const { return daynum_offset_epoch; }
/// All functions below are thread-safe; arguments are not checked.
inline ExtendedDayNum toDayNum(ExtendedDayNum d) const
@ -926,15 +930,17 @@ public:
{
if (unlikely(year < DATE_LUT_MIN_YEAR || year > DATE_LUT_MAX_YEAR || month < 1 || month > 12 || day_of_month < 1 || day_of_month > 31))
return LUTIndex(0);
return LUTIndex{years_months_lut[(year - DATE_LUT_MIN_YEAR) * 12 + month - 1] + day_of_month - 1};
auto year_lut_index = (year - DATE_LUT_MIN_YEAR) * 12 + month - 1;
UInt32 index = years_months_lut[year_lut_index].toUnderType() + day_of_month - 1;
/// When date is out of range, default value is DATE_LUT_SIZE - 1 (2283-11-11)
return LUTIndex{std::min(index, static_cast<UInt32>(DATE_LUT_SIZE - 1))};
}
/// Create DayNum from year, month, day of month.
inline ExtendedDayNum makeDayNum(Int16 year, UInt8 month, UInt8 day_of_month) const
inline ExtendedDayNum makeDayNum(Int16 year, UInt8 month, UInt8 day_of_month, Int32 default_error_day_num = 0) const
{
if (unlikely(year < DATE_LUT_MIN_YEAR || year > DATE_LUT_MAX_YEAR || month < 1 || month > 12 || day_of_month < 1 || day_of_month > 31))
return ExtendedDayNum(0);
return ExtendedDayNum(default_error_day_num);
return toDayNum(makeLUTIndex(year, month, day_of_month));
}
@ -1091,9 +1097,9 @@ public:
return lut[new_index].date + time;
}
inline NO_SANITIZE_UNDEFINED Time addWeeks(Time t, Int64 delta) const
inline NO_SANITIZE_UNDEFINED Time addWeeks(Time t, Int32 delta) const
{
return addDays(t, delta * 7);
return addDays(t, static_cast<Int64>(delta) * 7);
}
inline UInt8 saturateDayOfMonth(Int16 year, UInt8 month, UInt8 day_of_month) const
@ -1158,14 +1164,14 @@ public:
return toDayNum(addMonthsIndex(d, delta));
}
inline Time NO_SANITIZE_UNDEFINED addQuarters(Time t, Int64 delta) const
inline Time NO_SANITIZE_UNDEFINED addQuarters(Time t, Int32 delta) const
{
return addMonths(t, delta * 3);
return addMonths(t, static_cast<Int64>(delta) * 3);
}
inline ExtendedDayNum addQuarters(ExtendedDayNum d, Int64 delta) const
inline ExtendedDayNum addQuarters(ExtendedDayNum d, Int32 delta) const
{
return addMonths(d, delta * 3);
return addMonths(d, static_cast<Int64>(delta) * 3);
}
template <typename DateOrTime>

View File

@ -70,6 +70,14 @@ public:
m_day = values.day_of_month;
}
explicit LocalDate(ExtendedDayNum day_num)
{
const auto & values = DateLUT::instance().getValues(day_num);
m_year = values.year;
m_month = values.month;
m_day = values.day_of_month;
}
LocalDate(unsigned short year_, unsigned char month_, unsigned char day_)
: m_year(year_), m_month(month_), m_day(day_)
{
@ -98,6 +106,12 @@ public:
return DayNum(lut.makeDayNum(m_year, m_month, m_day).toUnderType());
}
ExtendedDayNum getExtenedDayNum() const
{
const auto & lut = DateLUT::instance();
return ExtendedDayNum (lut.makeDayNum(m_year, m_month, m_day).toUnderType());
}
operator DayNum() const
{
return getDayNum();

View File

@ -115,6 +115,8 @@ set(S2_SRCS
add_library(s2 ${S2_SRCS})
set_property(TARGET s2 PROPERTY CXX_STANDARD 11)
if (OPENSSL_FOUND)
target_link_libraries(s2 PRIVATE ${OPENSSL_LIBRARIES})
endif()

View File

@ -27,7 +27,7 @@ RUN apt-get update \
# Special dpkg-deb (https://github.com/ClickHouse-Extras/dpkg) version which is able
# to compress files using pigz (https://zlib.net/pigz/) instead of gzip.
# Significantly increase deb packaging speed and compatible with old systems
RUN curl -O https://clickhouse-builds.s3.yandex.net/utils/1/dpkg-deb \
RUN curl -O https://clickhouse-datasets.s3.yandex.net/utils/1/dpkg-deb \
&& chmod +x dpkg-deb \
&& cp dpkg-deb /usr/bin

View File

@ -2,7 +2,7 @@
FROM yandex/clickhouse-deb-builder
RUN export CODENAME="$(lsb_release --codename --short | tr 'A-Z' 'a-z')" \
&& wget -nv -O /tmp/arrow-keyring.deb "https://apache.bintray.com/arrow/ubuntu/apache-arrow-archive-keyring-latest-${CODENAME}.deb" \
&& wget -nv -O /tmp/arrow-keyring.deb "https://apache.jfrog.io/artifactory/arrow/ubuntu/apache-arrow-apt-source-latest-${CODENAME}.deb" \
&& dpkg -i /tmp/arrow-keyring.deb
# Libraries from OS are only needed to test the "unbundled" build (that is not used in production).

View File

@ -27,7 +27,7 @@ RUN apt-get update \
# Special dpkg-deb (https://github.com/ClickHouse-Extras/dpkg) version which is able
# to compress files using pigz (https://zlib.net/pigz/) instead of gzip.
# Significantly increase deb packaging speed and compatible with old systems
RUN curl -O https://clickhouse-builds.s3.yandex.net/utils/1/dpkg-deb \
RUN curl -O https://clickhouse-datasets.s3.yandex.net/utils/1/dpkg-deb \
&& chmod +x dpkg-deb \
&& cp dpkg-deb /usr/bin

View File

@ -27,7 +27,7 @@ RUN apt-get update \
# Special dpkg-deb (https://github.com/ClickHouse-Extras/dpkg) version which is able
# to compress files using pigz (https://zlib.net/pigz/) instead of gzip.
# Significantly increase deb packaging speed and compatible with old systems
RUN curl -O https://clickhouse-builds.s3.yandex.net/utils/1/dpkg-deb \
RUN curl -O https://clickhouse-datasets.s3.yandex.net/utils/1/dpkg-deb \
&& chmod +x dpkg-deb \
&& cp dpkg-deb /usr/bin

View File

@ -123,7 +123,7 @@ For installing CMake and Ninja on Mac OS X first install Homebrew and then insta
/usr/bin/ruby -e "$(curl -fsSL https://raw.githubusercontent.com/Homebrew/install/master/install)"
brew install cmake ninja
Next, check the version of CMake: `cmake --version`. If it is below 3.3, you should install a newer version from the website: https://cmake.org/download/.
Next, check the version of CMake: `cmake --version`. If it is below 3.12, you should install a newer version from the website: https://cmake.org/download/.
## Optional External Libraries {#optional-external-libraries}

View File

@ -82,6 +82,8 @@ MySQL DDL queries are converted into the corresponding ClickHouse DDL queries ([
- If `_sign` is not specified in the `SELECT` query, `WHERE _sign=1` is used by default. So the deleted rows are not included into the result set.
- The result includes columns comments in case they exist in MySQL database tables.
### Index Conversion {#index-conversion}
MySQL `PRIMARY KEY` and `INDEX` clauses are converted into `ORDER BY` tuples in ClickHouse tables.

View File

@ -189,7 +189,7 @@ CREATE TABLE codec_example
dt Date CODEC(ZSTD),
ts DateTime CODEC(LZ4HC),
float_value Float32 CODEC(NONE),
double_value Float64 CODEC(LZ4HC(9))
double_value Float64 CODEC(LZ4HC(9)),
value Float32 CODEC(Delta, ZSTD)
)
ENGINE = <Engine>

View File

@ -128,7 +128,7 @@ Ninja - система запуска сборочных задач.
/usr/bin/ruby -e "$(curl -fsSL https://raw.githubusercontent.com/Homebrew/install/master/install)"
brew install cmake ninja
Проверьте версию CMake: `cmake --version`. Если версия меньше 3.3, то установите новую версию с сайта https://cmake.org/download/
Проверьте версию CMake: `cmake --version`. Если версия меньше 3.12, то установите новую версию с сайта https://cmake.org/download/
## Необязательные внешние библиотеки {#neobiazatelnye-vneshnie-biblioteki}

View File

@ -1,3 +1,4 @@
---
toc_priority: 29
toc_title: MaterializeMySQL
@ -80,7 +81,9 @@ DDL-запросы в MySQL конвертируются в соответств
- Если в запросе `SELECT` напрямую не указан столбец `_version`, то используется модификатор [FINAL](../../sql-reference/statements/select/from.md#select-from-final). Таким образом, выбираются только строки с `MAX(_version)`.
- Если в запросе `SELECT` напрямую не указан столбец `_sign`, то по умолчанию используется `WHERE _sign=1`. Таким образом, удаленные строки не включаются в результирующий набор.
- Если в запросе `SELECT` напрямую не указан столбец `_sign`, то по умолчанию используется `WHERE _sign=1`. Таким образом, удаленные строки не включаются в результирующий набор.
- Результат включает комментарии к столбцам, если они существуют в таблицах базы данных MySQL.
### Конвертация индексов {#index-conversion}

View File

@ -433,7 +433,7 @@ void LocalServer::processQueries()
try
{
executeQuery(read_buf, write_buf, /* allow_into_outfile = */ true, context, {}, finalize_progress);
executeQuery(read_buf, write_buf, /* allow_into_outfile = */ true, context, {}, {}, finalize_progress);
}
catch (...)
{

View File

@ -3,6 +3,7 @@
#include <AggregateFunctions/AggregateFunctionSequenceMatch.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <common/range.h>

View File

@ -459,6 +459,8 @@ public:
explicit FieldVisitorMax(const Field & rhs_) : rhs(rhs_) {}
bool operator() (Null &) const { throw Exception("Cannot compare Nulls", ErrorCodes::LOGICAL_ERROR); }
bool operator() (NegativeInfinity &) const { throw Exception("Cannot compare -Inf", ErrorCodes::LOGICAL_ERROR); }
bool operator() (PositiveInfinity &) const { throw Exception("Cannot compare +Inf", ErrorCodes::LOGICAL_ERROR); }
bool operator() (AggregateFunctionStateData &) const { throw Exception("Cannot compare AggregateFunctionStates", ErrorCodes::LOGICAL_ERROR); }
bool operator() (Array & x) const { return compareImpl<Array>(x); }
@ -494,6 +496,8 @@ public:
explicit FieldVisitorMin(const Field & rhs_) : rhs(rhs_) {}
bool operator() (Null &) const { throw Exception("Cannot compare Nulls", ErrorCodes::LOGICAL_ERROR); }
bool operator() (NegativeInfinity &) const { throw Exception("Cannot compare -Inf", ErrorCodes::LOGICAL_ERROR); }
bool operator() (PositiveInfinity &) const { throw Exception("Cannot compare +Inf", ErrorCodes::LOGICAL_ERROR); }
bool operator() (AggregateFunctionStateData &) const { throw Exception("Cannot sum AggregateFunctionStates", ErrorCodes::LOGICAL_ERROR); }
bool operator() (Array & x) const { return compareImpl<Array>(x); }

View File

@ -4,6 +4,7 @@
#include <AggregateFunctions/FactoryHelpers.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeTuple.h>
#include <DataTypes/DataTypeUUID.h>
@ -49,6 +50,8 @@ AggregateFunctionPtr createAggregateFunctionUniq(const std::string & name, const
return res;
else if (which.isDate())
return std::make_shared<AggregateFunctionUniq<DataTypeDate::FieldType, Data>>(argument_types);
else if (which.isDate32())
return std::make_shared<AggregateFunctionUniq<DataTypeDate32::FieldType, Data>>(argument_types);
else if (which.isDateTime())
return std::make_shared<AggregateFunctionUniq<DataTypeDateTime::FieldType, Data>>(argument_types);
else if (which.isStringOrFixedString())
@ -95,6 +98,8 @@ AggregateFunctionPtr createAggregateFunctionUniq(const std::string & name, const
return res;
else if (which.isDate())
return std::make_shared<AggregateFunctionUniq<DataTypeDate::FieldType, Data<DataTypeDate::FieldType>>>(argument_types);
else if (which.isDate32())
return std::make_shared<AggregateFunctionUniq<DataTypeDate32::FieldType, Data<DataTypeDate32::FieldType>>>(argument_types);
else if (which.isDateTime())
return std::make_shared<AggregateFunctionUniq<DataTypeDateTime::FieldType, Data<DataTypeDateTime::FieldType>>>(argument_types);
else if (which.isStringOrFixedString())

View File

@ -6,6 +6,7 @@
#include <Common/FieldVisitorConvertToNumber.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <functional>
@ -51,6 +52,8 @@ namespace
return res;
else if (which.isDate())
return std::make_shared<typename WithK<K, HashValueType>::template AggregateFunction<DataTypeDate::FieldType>>(argument_types, params);
else if (which.isDate32())
return std::make_shared<typename WithK<K, HashValueType>::template AggregateFunction<DataTypeDate32::FieldType>>(argument_types, params);
else if (which.isDateTime())
return std::make_shared<typename WithK<K, HashValueType>::template AggregateFunction<DataTypeDateTime::FieldType>>(argument_types, params);
else if (which.isStringOrFixedString())

View File

@ -3,6 +3,7 @@
#include <AggregateFunctions/AggregateFunctionUniqUpTo.h>
#include <Common/FieldVisitorConvertToNumber.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeString.h>
#include <DataTypes/DataTypeFixedString.h>
@ -61,6 +62,8 @@ AggregateFunctionPtr createAggregateFunctionUniqUpTo(const std::string & name, c
return res;
else if (which.isDate())
return std::make_shared<AggregateFunctionUniqUpTo<DataTypeDate::FieldType>>(threshold, argument_types, params);
else if (which.isDate32())
return std::make_shared<AggregateFunctionUniqUpTo<DataTypeDate32::FieldType>>(threshold, argument_types, params);
else if (which.isDateTime())
return std::make_shared<AggregateFunctionUniqUpTo<DataTypeDateTime::FieldType>>(threshold, argument_types, params);
else if (which.isStringOrFixedString())

View File

@ -4,6 +4,7 @@
#include <AggregateFunctions/Helpers.h>
#include <Core/Settings.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <common/range.h>

View File

@ -546,97 +546,54 @@ namespace
{
/// The following function implements a slightly more general version
/// of getExtremes() than the implementation from ColumnVector.
/// of getExtremes() than the implementation from Not-Null IColumns.
/// It takes into account the possible presence of nullable values.
template <typename T>
void getExtremesFromNullableContent(const ColumnVector<T> & col, const NullMap & null_map, Field & min, Field & max)
void getExtremesWithNulls(const IColumn & nested_column, const NullMap & null_array, Field & min, Field & max, bool null_last = false)
{
const auto & data = col.getData();
size_t size = data.size();
if (size == 0)
size_t number_of_nulls = 0;
size_t n = null_array.size();
NullMap not_null_array(n);
for (auto i = 0ul; i < n; ++i)
{
min = Null();
max = Null();
return;
}
bool has_not_null = false;
bool has_not_nan = false;
T cur_min = 0;
T cur_max = 0;
for (size_t i = 0; i < size; ++i)
{
const T x = data[i];
if (null_map[i])
continue;
if (!has_not_null)
if (null_array[i])
{
cur_min = x;
cur_max = x;
has_not_null = true;
has_not_nan = !isNaN(x);
continue;
++number_of_nulls;
not_null_array[i] = 0;
}
if (isNaN(x))
continue;
if (!has_not_nan)
else
{
cur_min = x;
cur_max = x;
has_not_nan = true;
continue;
not_null_array[i] = 1;
}
if (x < cur_min)
cur_min = x;
else if (x > cur_max)
cur_max = x;
}
if (has_not_null)
if (number_of_nulls == 0)
{
min = cur_min;
max = cur_max;
nested_column.getExtremes(min, max);
}
else if (number_of_nulls == n)
{
min = PositiveInfinity();
max = PositiveInfinity();
}
else
{
auto filtered_column = nested_column.filter(not_null_array, -1);
filtered_column->getExtremes(min, max);
if (null_last)
max = PositiveInfinity();
}
}
}
void ColumnNullable::getExtremes(Field & min, Field & max) const
{
min = Null();
max = Null();
getExtremesWithNulls(getNestedColumn(), getNullMapData(), min, max);
}
const auto & null_map_data = getNullMapData();
if (const auto * col_i8 = typeid_cast<const ColumnInt8 *>(nested_column.get()))
getExtremesFromNullableContent<Int8>(*col_i8, null_map_data, min, max);
else if (const auto * col_i16 = typeid_cast<const ColumnInt16 *>(nested_column.get()))
getExtremesFromNullableContent<Int16>(*col_i16, null_map_data, min, max);
else if (const auto * col_i32 = typeid_cast<const ColumnInt32 *>(nested_column.get()))
getExtremesFromNullableContent<Int32>(*col_i32, null_map_data, min, max);
else if (const auto * col_i64 = typeid_cast<const ColumnInt64 *>(nested_column.get()))
getExtremesFromNullableContent<Int64>(*col_i64, null_map_data, min, max);
else if (const auto * col_u8 = typeid_cast<const ColumnUInt8 *>(nested_column.get()))
getExtremesFromNullableContent<UInt8>(*col_u8, null_map_data, min, max);
else if (const auto * col_u16 = typeid_cast<const ColumnUInt16 *>(nested_column.get()))
getExtremesFromNullableContent<UInt16>(*col_u16, null_map_data, min, max);
else if (const auto * col_u32 = typeid_cast<const ColumnUInt32 *>(nested_column.get()))
getExtremesFromNullableContent<UInt32>(*col_u32, null_map_data, min, max);
else if (const auto * col_u64 = typeid_cast<const ColumnUInt64 *>(nested_column.get()))
getExtremesFromNullableContent<UInt64>(*col_u64, null_map_data, min, max);
else if (const auto * col_f32 = typeid_cast<const ColumnFloat32 *>(nested_column.get()))
getExtremesFromNullableContent<Float32>(*col_f32, null_map_data, min, max);
else if (const auto * col_f64 = typeid_cast<const ColumnFloat64 *>(nested_column.get()))
getExtremesFromNullableContent<Float64>(*col_f64, null_map_data, min, max);
void ColumnNullable::getExtremesNullLast(Field & min, Field & max) const
{
getExtremesWithNulls(getNestedColumn(), getNullMapData(), min, max, true);
}

View File

@ -111,6 +111,8 @@ public:
void updateWeakHash32(WeakHash32 & hash) const override;
void updateHashFast(SipHash & hash) const override;
void getExtremes(Field & min, Field & max) const override;
// Special function for nullable minmax index
void getExtremesNullLast(Field & min, Field & max) const;
MutableColumns scatter(ColumnIndex num_columns, const Selector & selector) const override
{

View File

@ -26,6 +26,16 @@ public:
throw Exception("Cannot convert NULL to " + demangle(typeid(T).name()), ErrorCodes::CANNOT_CONVERT_TYPE);
}
T operator() (const NegativeInfinity &) const
{
throw Exception("Cannot convert -Inf to " + demangle(typeid(T).name()), ErrorCodes::CANNOT_CONVERT_TYPE);
}
T operator() (const PositiveInfinity &) const
{
throw Exception("Cannot convert +Inf to " + demangle(typeid(T).name()), ErrorCodes::CANNOT_CONVERT_TYPE);
}
T operator() (const String &) const
{
throw Exception("Cannot convert String to " + demangle(typeid(T).name()), ErrorCodes::CANNOT_CONVERT_TYPE);

View File

@ -25,6 +25,8 @@ static inline void writeQuoted(const DecimalField<T> & x, WriteBuffer & buf)
}
String FieldVisitorDump::operator() (const Null &) const { return "NULL"; }
String FieldVisitorDump::operator() (const NegativeInfinity &) const { return "-Inf"; }
String FieldVisitorDump::operator() (const PositiveInfinity &) const { return "+Inf"; }
String FieldVisitorDump::operator() (const UInt64 & x) const { return formatQuotedWithPrefix(x, "UInt64_"); }
String FieldVisitorDump::operator() (const Int64 & x) const { return formatQuotedWithPrefix(x, "Int64_"); }
String FieldVisitorDump::operator() (const Float64 & x) const { return formatQuotedWithPrefix(x, "Float64_"); }

View File

@ -10,6 +10,8 @@ class FieldVisitorDump : public StaticVisitor<String>
{
public:
String operator() (const Null & x) const;
String operator() (const NegativeInfinity & x) const;
String operator() (const PositiveInfinity & x) const;
String operator() (const UInt64 & x) const;
String operator() (const UInt128 & x) const;
String operator() (const UInt256 & x) const;

View File

@ -14,6 +14,18 @@ void FieldVisitorHash::operator() (const Null &) const
hash.update(type);
}
void FieldVisitorHash::operator() (const NegativeInfinity &) const
{
UInt8 type = Field::Types::NegativeInfinity;
hash.update(type);
}
void FieldVisitorHash::operator() (const PositiveInfinity &) const
{
UInt8 type = Field::Types::PositiveInfinity;
hash.update(type);
}
void FieldVisitorHash::operator() (const UInt64 & x) const
{
UInt8 type = Field::Types::UInt64;

View File

@ -16,6 +16,8 @@ public:
FieldVisitorHash(SipHash & hash_);
void operator() (const Null & x) const;
void operator() (const NegativeInfinity & x) const;
void operator() (const PositiveInfinity & x) const;
void operator() (const UInt64 & x) const;
void operator() (const UInt128 & x) const;
void operator() (const UInt256 & x) const;

View File

@ -22,6 +22,8 @@ bool FieldVisitorSum::operator() (UInt64 & x) const
bool FieldVisitorSum::operator() (Float64 & x) const { x += get<Float64>(rhs); return x != 0; }
bool FieldVisitorSum::operator() (Null &) const { throw Exception("Cannot sum Nulls", ErrorCodes::LOGICAL_ERROR); }
bool FieldVisitorSum::operator() (NegativeInfinity &) const { throw Exception("Cannot sum -Inf", ErrorCodes::LOGICAL_ERROR); }
bool FieldVisitorSum::operator() (PositiveInfinity &) const { throw Exception("Cannot sum +Inf", ErrorCodes::LOGICAL_ERROR); }
bool FieldVisitorSum::operator() (String &) const { throw Exception("Cannot sum Strings", ErrorCodes::LOGICAL_ERROR); }
bool FieldVisitorSum::operator() (Array &) const { throw Exception("Cannot sum Arrays", ErrorCodes::LOGICAL_ERROR); }
bool FieldVisitorSum::operator() (Tuple &) const { throw Exception("Cannot sum Tuples", ErrorCodes::LOGICAL_ERROR); }

View File

@ -21,6 +21,8 @@ public:
bool operator() (UInt64 & x) const;
bool operator() (Float64 & x) const;
bool operator() (Null &) const;
bool operator() (NegativeInfinity & x) const;
bool operator() (PositiveInfinity & x) const;
bool operator() (String &) const;
bool operator() (Array &) const;
bool operator() (Tuple &) const;

View File

@ -53,6 +53,8 @@ static String formatFloat(const Float64 x)
String FieldVisitorToString::operator() (const Null &) const { return "NULL"; }
String FieldVisitorToString::operator() (const NegativeInfinity &) const { return "-Inf"; }
String FieldVisitorToString::operator() (const PositiveInfinity &) const { return "+Inf"; }
String FieldVisitorToString::operator() (const UInt64 & x) const { return formatQuoted(x); }
String FieldVisitorToString::operator() (const Int64 & x) const { return formatQuoted(x); }
String FieldVisitorToString::operator() (const Float64 & x) const { return formatFloat(x); }

View File

@ -10,6 +10,8 @@ class FieldVisitorToString : public StaticVisitor<String>
{
public:
String operator() (const Null & x) const;
String operator() (const NegativeInfinity & x) const;
String operator() (const PositiveInfinity & x) const;
String operator() (const UInt64 & x) const;
String operator() (const UInt128 & x) const;
String operator() (const UInt256 & x) const;

View File

@ -7,6 +7,8 @@ namespace DB
{
void FieldVisitorWriteBinary::operator() (const Null &, WriteBuffer &) const { }
void FieldVisitorWriteBinary::operator() (const NegativeInfinity &, WriteBuffer &) const { }
void FieldVisitorWriteBinary::operator() (const PositiveInfinity &, WriteBuffer &) const { }
void FieldVisitorWriteBinary::operator() (const UInt64 & x, WriteBuffer & buf) const { writeVarUInt(x, buf); }
void FieldVisitorWriteBinary::operator() (const Int64 & x, WriteBuffer & buf) const { writeVarInt(x, buf); }
void FieldVisitorWriteBinary::operator() (const Float64 & x, WriteBuffer & buf) const { writeFloatBinary(x, buf); }

View File

@ -9,6 +9,8 @@ class FieldVisitorWriteBinary
{
public:
void operator() (const Null & x, WriteBuffer & buf) const;
void operator() (const NegativeInfinity & x, WriteBuffer & buf) const;
void operator() (const PositiveInfinity & x, WriteBuffer & buf) const;
void operator() (const UInt64 & x, WriteBuffer & buf) const;
void operator() (const UInt128 & x, WriteBuffer & buf) const;
void operator() (const UInt256 & x, WriteBuffer & buf) const;

View File

@ -26,8 +26,12 @@ public:
template <typename T, typename U>
bool operator() (const T & l, const U & r) const
{
if constexpr (std::is_same_v<T, Null> || std::is_same_v<U, Null>)
if constexpr (std::is_same_v<T, Null> || std::is_same_v<U, Null>
|| std::is_same_v<T, NegativeInfinity> || std::is_same_v<T, PositiveInfinity>
|| std::is_same_v<U, NegativeInfinity> || std::is_same_v<U, PositiveInfinity>)
{
return std::is_same_v<T, U>;
}
else
{
if constexpr (std::is_same_v<T, U>)
@ -77,6 +81,10 @@ public:
{
if constexpr (std::is_same_v<T, Null> || std::is_same_v<U, Null>)
return false;
else if constexpr (std::is_same_v<T, NegativeInfinity> || std::is_same_v<U, PositiveInfinity>)
return !std::is_same_v<T, U>;
else if constexpr (std::is_same_v<U, NegativeInfinity> || std::is_same_v<T, PositiveInfinity>)
return false;
else
{
if constexpr (std::is_same_v<T, U>)

View File

@ -62,6 +62,8 @@ void ExternalResultDescription::init(const Block & sample_block_)
types.emplace_back(ValueType::vtString, is_nullable);
else if (which.isDate())
types.emplace_back(ValueType::vtDate, is_nullable);
else if (which.isDate32())
types.emplace_back(ValueType::vtDate32, is_nullable);
else if (which.isDateTime())
types.emplace_back(ValueType::vtDateTime, is_nullable);
else if (which.isUUID())

View File

@ -26,6 +26,7 @@ struct ExternalResultDescription
vtEnum16,
vtString,
vtDate,
vtDate32,
vtDateTime,
vtUUID,
vtDateTime64,

View File

@ -455,6 +455,16 @@ inline void writeText(const Null &, WriteBuffer & buf)
writeText(std::string("NULL"), buf);
}
inline void writeText(const NegativeInfinity &, WriteBuffer & buf)
{
writeText(std::string("-Inf"), buf);
}
inline void writeText(const PositiveInfinity &, WriteBuffer & buf)
{
writeText(std::string("+Inf"), buf);
}
String toString(const Field & x)
{
return Field::dispatch(

View File

@ -218,6 +218,8 @@ template <> struct NearestFieldTypeImpl<Tuple> { using Type = Tuple; };
template <> struct NearestFieldTypeImpl<Map> { using Type = Map; };
template <> struct NearestFieldTypeImpl<bool> { using Type = UInt64; };
template <> struct NearestFieldTypeImpl<Null> { using Type = Null; };
template <> struct NearestFieldTypeImpl<NegativeInfinity> { using Type = NegativeInfinity; };
template <> struct NearestFieldTypeImpl<PositiveInfinity> { using Type = PositiveInfinity; };
template <> struct NearestFieldTypeImpl<AggregateFunctionStateData> { using Type = AggregateFunctionStateData; };
@ -269,6 +271,10 @@ public:
Int256 = 25,
Map = 26,
UUID = 27,
// Special types for index analysis
NegativeInfinity = 254,
PositiveInfinity = 255,
};
static const char * toString(Which which)
@ -276,6 +282,8 @@ public:
switch (which)
{
case Null: return "Null";
case NegativeInfinity: return "-Inf";
case PositiveInfinity: return "+Inf";
case UInt64: return "UInt64";
case UInt128: return "UInt128";
case UInt256: return "UInt256";
@ -404,7 +412,10 @@ public:
Types::Which getType() const { return which; }
const char * getTypeName() const { return Types::toString(which); }
bool isNull() const { return which == Types::Null; }
// Non-valued field are all denoted as Null
bool isNull() const { return which == Types::Null || which == Types::NegativeInfinity || which == Types::PositiveInfinity; }
bool isNegativeInfinity() const { return which == Types::NegativeInfinity; }
bool isPositiveInfinity() const { return which == Types::PositiveInfinity; }
template <typename T>
@ -459,7 +470,10 @@ public:
switch (which)
{
case Types::Null: return false;
case Types::Null:
case Types::NegativeInfinity:
case Types::PositiveInfinity:
return false;
case Types::UInt64: return get<UInt64>() < rhs.get<UInt64>();
case Types::UInt128: return get<UInt128>() < rhs.get<UInt128>();
case Types::UInt256: return get<UInt256>() < rhs.get<UInt256>();
@ -496,7 +510,10 @@ public:
switch (which)
{
case Types::Null: return true;
case Types::Null:
case Types::NegativeInfinity:
case Types::PositiveInfinity:
return true;
case Types::UInt64: return get<UInt64>() <= rhs.get<UInt64>();
case Types::UInt128: return get<UInt128>() <= rhs.get<UInt128>();
case Types::UInt256: return get<UInt256>() <= rhs.get<UInt256>();
@ -533,8 +550,11 @@ public:
switch (which)
{
case Types::Null: return true;
case Types::UInt64: return get<UInt64>() == rhs.get<UInt64>();
case Types::Null:
case Types::NegativeInfinity:
case Types::PositiveInfinity:
return true;
case Types::UInt64: return get<UInt64>() == rhs.get<UInt64>();
case Types::Int64: return get<Int64>() == rhs.get<Int64>();
case Types::Float64:
{
@ -573,6 +593,8 @@ public:
switch (field.which)
{
case Types::Null: return f(field.template get<Null>());
case Types::NegativeInfinity: return f(field.template get<NegativeInfinity>());
case Types::PositiveInfinity: return f(field.template get<PositiveInfinity>());
// gcc 8.2.1
#if !defined(__clang__)
#pragma GCC diagnostic push
@ -731,6 +753,8 @@ using Row = std::vector<Field>;
template <> struct Field::TypeToEnum<Null> { static const Types::Which value = Types::Null; };
template <> struct Field::TypeToEnum<NegativeInfinity> { static const Types::Which value = Types::NegativeInfinity; };
template <> struct Field::TypeToEnum<PositiveInfinity> { static const Types::Which value = Types::PositiveInfinity; };
template <> struct Field::TypeToEnum<UInt64> { static const Types::Which value = Types::UInt64; };
template <> struct Field::TypeToEnum<UInt128> { static const Types::Which value = Types::UInt128; };
template <> struct Field::TypeToEnum<UInt256> { static const Types::Which value = Types::UInt256; };
@ -751,6 +775,8 @@ template <> struct Field::TypeToEnum<DecimalField<DateTime64>>{ static const Typ
template <> struct Field::TypeToEnum<AggregateFunctionStateData>{ static const Types::Which value = Types::AggregateFunctionState; };
template <> struct Field::EnumToType<Field::Types::Null> { using Type = Null; };
template <> struct Field::EnumToType<Field::Types::NegativeInfinity> { using Type = NegativeInfinity; };
template <> struct Field::EnumToType<Field::Types::PositiveInfinity> { using Type = PositiveInfinity; };
template <> struct Field::EnumToType<Field::Types::UInt64> { using Type = UInt64; };
template <> struct Field::EnumToType<Field::Types::UInt128> { using Type = UInt128; };
template <> struct Field::EnumToType<Field::Types::UInt256> { using Type = UInt256; };

View File

@ -24,16 +24,15 @@ namespace ErrorCodes
}
MySQLClient::MySQLClient(const String & host_, UInt16 port_, const String & user_, const String & password_)
: host(host_), port(port_), user(user_), password(std::move(password_))
: host(host_), port(port_), user(user_), password(std::move(password_)),
client_capabilities(CLIENT_PROTOCOL_41 | CLIENT_PLUGIN_AUTH | CLIENT_SECURE_CONNECTION)
{
mysql_context.client_capabilities = CLIENT_PROTOCOL_41 | CLIENT_PLUGIN_AUTH | CLIENT_SECURE_CONNECTION;
}
MySQLClient::MySQLClient(MySQLClient && other)
: host(std::move(other.host)), port(other.port), user(std::move(other.user)), password(std::move(other.password))
, mysql_context(other.mysql_context)
, client_capabilities(other.client_capabilities)
{
mysql_context.sequence_id = 0;
}
void MySQLClient::connect()
@ -57,7 +56,8 @@ void MySQLClient::connect()
in = std::make_shared<ReadBufferFromPocoSocket>(*socket);
out = std::make_shared<WriteBufferFromPocoSocket>(*socket);
packet_endpoint = mysql_context.makeEndpoint(*in, *out);
packet_endpoint = MySQLProtocol::PacketEndpoint::create(*in, *out, sequence_id);
handshake();
}
@ -69,7 +69,7 @@ void MySQLClient::disconnect()
socket->close();
socket = nullptr;
connected = false;
mysql_context.sequence_id = 0;
sequence_id = 0;
}
/// https://dev.mysql.com/doc/internals/en/connection-phase-packets.html
@ -88,10 +88,10 @@ void MySQLClient::handshake()
String auth_plugin_data = native41.getAuthPluginData();
HandshakeResponse handshake_response(
mysql_context.client_capabilities, MAX_PACKET_LENGTH, charset_utf8, user, "", auth_plugin_data, mysql_native_password);
client_capabilities, MAX_PACKET_LENGTH, charset_utf8, user, "", auth_plugin_data, mysql_native_password);
packet_endpoint->sendPacket<HandshakeResponse>(handshake_response, true);
ResponsePacket packet_response(mysql_context.client_capabilities, true);
ResponsePacket packet_response(client_capabilities, true);
packet_endpoint->receivePacket(packet_response);
packet_endpoint->resetSequenceId();
@ -106,7 +106,7 @@ void MySQLClient::writeCommand(char command, String query)
WriteCommand write_command(command, query);
packet_endpoint->sendPacket<WriteCommand>(write_command, true);
ResponsePacket packet_response(mysql_context.client_capabilities);
ResponsePacket packet_response(client_capabilities);
packet_endpoint->receivePacket(packet_response);
switch (packet_response.getType())
{
@ -125,7 +125,7 @@ void MySQLClient::registerSlaveOnMaster(UInt32 slave_id)
RegisterSlave register_slave(slave_id);
packet_endpoint->sendPacket<RegisterSlave>(register_slave, true);
ResponsePacket packet_response(mysql_context.client_capabilities);
ResponsePacket packet_response(client_capabilities);
packet_endpoint->receivePacket(packet_response);
packet_endpoint->resetSequenceId();
if (packet_response.getType() == PACKET_ERR)

View File

@ -45,7 +45,9 @@ private:
String password;
bool connected = false;
MySQLWireContext mysql_context;
uint8_t sequence_id = 0;
uint32_t client_capabilities = 0;
const UInt8 charset_utf8 = 33;
const String mysql_native_password = "mysql_native_password";

View File

@ -68,15 +68,4 @@ String PacketEndpoint::packetToText(const String & payload)
}
MySQLProtocol::PacketEndpointPtr MySQLWireContext::makeEndpoint(WriteBuffer & out)
{
return MySQLProtocol::PacketEndpoint::create(out, sequence_id);
}
MySQLProtocol::PacketEndpointPtr MySQLWireContext::makeEndpoint(ReadBuffer & in, WriteBuffer & out)
{
return MySQLProtocol::PacketEndpoint::create(in, out, sequence_id);
}
}

View File

@ -58,14 +58,4 @@ using PacketEndpointPtr = std::shared_ptr<PacketEndpoint>;
}
struct MySQLWireContext
{
uint8_t sequence_id = 0;
uint32_t client_capabilities = 0;
size_t max_packet_size = 0;
MySQLProtocol::PacketEndpointPtr makeEndpoint(WriteBuffer & out);
MySQLProtocol::PacketEndpointPtr makeEndpoint(ReadBuffer & in, WriteBuffer & out);
};
}

View File

@ -90,6 +90,9 @@ void insertPostgreSQLValue(
case ExternalResultDescription::ValueType::vtDate:
assert_cast<ColumnUInt16 &>(column).insertValue(UInt16{LocalDate{std::string(value)}.getDayNum()});
break;
case ExternalResultDescription::ValueType::vtDate32:
assert_cast<ColumnInt32 &>(column).insertValue(Int32{LocalDate{std::string(value)}.getExtenedDayNum()});
break;
case ExternalResultDescription::ValueType::vtDateTime:
{
ReadBufferFromString in(value);

View File

@ -7,7 +7,6 @@
#if USE_LIBPQXX
#include <Core/Block.h>
#include <DataStreams/IBlockInputStream.h>
#include <Core/ExternalResultDescription.h>
#include <Core/Field.h>

View File

@ -14,6 +14,8 @@ namespace DB
/// Data types for representing elementary values from a database in RAM.
struct Null {};
struct NegativeInfinity {};
struct PositiveInfinity {};
/// Ignore strange gcc warning https://gcc.gnu.org/bugzilla/show_bug.cgi?id=55776
#if !defined(__clang__)
@ -39,6 +41,7 @@ enum class TypeIndex
Float32,
Float64,
Date,
Date32,
DateTime,
DateTime64,
String,
@ -257,6 +260,7 @@ inline constexpr const char * getTypeName(TypeIndex idx)
case TypeIndex::Float32: return "Float32";
case TypeIndex::Float64: return "Float64";
case TypeIndex::Date: return "Date";
case TypeIndex::Date32: return "Date32";
case TypeIndex::DateTime: return "DateTime";
case TypeIndex::DateTime64: return "DateTime64";
case TypeIndex::String: return "String";

View File

@ -73,6 +73,7 @@ bool callOnBasicType(TypeIndex number, F && f)
switch (number)
{
case TypeIndex::Date: return f(TypePair<T, UInt16>());
case TypeIndex::Date32: return f(TypePair<T, Int32>());
case TypeIndex::DateTime: return f(TypePair<T, UInt32>());
case TypeIndex::DateTime64: return f(TypePair<T, DateTime64>());
default:
@ -142,6 +143,7 @@ inline bool callOnBasicTypes(TypeIndex type_num1, TypeIndex type_num2, F && f)
switch (type_num1)
{
case TypeIndex::Date: return callOnBasicType<UInt16, _int, _float, _decimal, _datetime>(type_num2, std::forward<F>(f));
case TypeIndex::Date32: return callOnBasicType<Int32, _int, _float, _decimal, _datetime>(type_num2, std::forward<F>(f));
case TypeIndex::DateTime: return callOnBasicType<UInt32, _int, _float, _decimal, _datetime>(type_num2, std::forward<F>(f));
case TypeIndex::DateTime64: return callOnBasicType<DateTime64, _int, _float, _decimal, _datetime>(type_num2, std::forward<F>(f));
default:
@ -154,6 +156,7 @@ inline bool callOnBasicTypes(TypeIndex type_num1, TypeIndex type_num2, F && f)
class DataTypeDate;
class DataTypeDate32;
class DataTypeString;
class DataTypeFixedString;
class DataTypeUUID;
@ -192,6 +195,7 @@ bool callOnIndexAndDataType(TypeIndex number, F && f, ExtraArgs && ... args)
case TypeIndex::Decimal256: return f(TypePair<DataTypeDecimal<Decimal256>, T>(), std::forward<ExtraArgs>(args)...);
case TypeIndex::Date: return f(TypePair<DataTypeDate, T>(), std::forward<ExtraArgs>(args)...);
case TypeIndex::Date32: return f(TypePair<DataTypeDate32, T>(), std::forward<ExtraArgs>(args)...);
case TypeIndex::DateTime: return f(TypePair<DataTypeDateTime, T>(), std::forward<ExtraArgs>(args)...);
case TypeIndex::DateTime64: return f(TypePair<DataTypeDateTime64, T>(), std::forward<ExtraArgs>(args)...);

View File

@ -6,7 +6,6 @@
#include <Core/ColumnWithTypeAndName.h>
#include <Core/Field.h>
#include <Core/NamesAndTypes.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataTypes/IDataType.h>
#include <Functions/IFunction.h>
#include <IO/WriteBufferFromOStream.h>
@ -28,12 +27,6 @@ std::ostream & operator<< <Field>(std::ostream & stream, const Field & what)
return stream;
}
std::ostream & operator<<(std::ostream & stream, const IBlockInputStream & what)
{
stream << "IBlockInputStream(name = " << what.getName() << ")";
return stream;
}
std::ostream & operator<<(std::ostream & stream, const NameAndTypePair & what)
{
stream << "NameAndTypePair(name = " << what.name << ", type = " << what.type << ")";

View File

@ -10,9 +10,6 @@ class Field;
template <typename T, typename U = std::enable_if_t<std::is_same_v<T, Field>>>
std::ostream & operator<<(std::ostream & stream, const T & what);
class IBlockInputStream;
std::ostream & operator<<(std::ostream & stream, const IBlockInputStream & what);
struct NameAndTypePair;
std::ostream & operator<<(std::ostream & stream, const NameAndTypePair & what);

View File

@ -13,6 +13,7 @@
#include <DataTypes/DataTypesNumber.h>
#include <DataTypes/DataTypesDecimal.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
#include <DataTypes/DataTypeEnum.h>

View File

@ -11,7 +11,6 @@ WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
See the License for the specific language governing permissions and
limitations under the License. */
#include <DataStreams/IBlockInputStream.h>
#include <Processors/Sources/SourceWithProgress.h>
#include <Processors/Transforms/AggregatingTransform.h>

View File

@ -1,6 +1,6 @@
#pragma once
#include <DataStreams/IBlockOutputStream.h>
#include <DataStreams/IBlockInputStream.h>
#include <Interpreters/ProcessList.h>

View File

@ -25,7 +25,6 @@ namespace ErrorCodes
extern const int TOO_MANY_BYTES;
extern const int TOO_MANY_ROWS_OR_BYTES;
extern const int LOGICAL_ERROR;
extern const int TOO_DEEP_PIPELINE;
}
@ -357,74 +356,4 @@ Block IBlockInputStream::getExtremes()
return res;
}
String IBlockInputStream::getTreeID() const
{
WriteBufferFromOwnString s;
s << getName();
if (!children.empty())
{
s << "(";
for (BlockInputStreams::const_iterator it = children.begin(); it != children.end(); ++it)
{
if (it != children.begin())
s << ", ";
s << (*it)->getTreeID();
}
s << ")";
}
return s.str();
}
size_t IBlockInputStream::checkDepthImpl(size_t max_depth, size_t level) const
{
if (children.empty())
return 0;
if (level > max_depth)
throw Exception("Query pipeline is too deep. Maximum: " + toString(max_depth), ErrorCodes::TOO_DEEP_PIPELINE);
size_t res = 0;
for (const auto & child : children)
{
size_t child_depth = child->checkDepth(level + 1);
if (child_depth > res)
res = child_depth;
}
return res + 1;
}
void IBlockInputStream::dumpTree(WriteBuffer & ostr, size_t indent, size_t multiplier) const
{
ostr << String(indent, ' ') << getName();
if (multiplier > 1)
ostr << " × " << multiplier;
//ostr << ": " << getHeader().dumpStructure();
ostr << '\n';
++indent;
/// If the subtree is repeated several times, then we output it once with the multiplier.
using Multipliers = std::map<String, size_t>;
Multipliers multipliers;
for (const auto & child : children)
++multipliers[child->getTreeID()];
for (const auto & child : children)
{
String id = child->getTreeID();
size_t & subtree_multiplier = multipliers[id];
if (subtree_multiplier != 0) /// Already printed subtrees are marked with zero in the array of multipliers.
{
child->dumpTree(ostr, indent, subtree_multiplier);
subtree_multiplier = 0;
}
}
}
}

View File

@ -23,15 +23,6 @@ namespace ErrorCodes
class ProcessListElement;
class EnabledQuota;
class QueryStatus;
struct SortColumnDescription;
using SortDescription = std::vector<SortColumnDescription>;
/** Callback to track the progress of the query.
* Used in IBlockInputStream and Context.
* The function takes the number of rows in the last block, the number of bytes in the last block.
* Note that the callback can be called from different threads.
*/
using ProgressCallback = std::function<void(const Progress & progress)>;
/** The stream interface for reading data by blocks from the database.
@ -93,15 +84,6 @@ public:
*/
virtual void readSuffix();
/// Must be called before `read()` and `readPrefix()`.
void dumpTree(WriteBuffer & ostr, size_t indent = 0, size_t multiplier = 1) const;
/** Check the depth of the pipeline.
* If max_depth is specified and the `depth` is greater - throw an exception.
* Must be called before `read()` and `readPrefix()`.
*/
size_t checkDepth(size_t max_depth) const { return checkDepthImpl(max_depth, max_depth); }
/// Do not allow to change the table while the blocks stream and its children are alive.
void addTableLock(const TableLockHolder & lock) { table_locks.push_back(lock); }
@ -269,9 +251,6 @@ private:
size_t checkDepthImpl(size_t max_depth, size_t level) const;
/// Get text with names of this source and the entire subtree.
String getTreeID() const;
template <typename F>
void forEachChild(F && f)
{

View File

@ -1,80 +0,0 @@
#pragma once
#include <DataStreams/IBlockInputStream.h>
namespace DB
{
/** Initialize another source on the first `read` call, and then use it.
* This is needed, for example, to read from a table that will be populated
* after creation of LazyBlockInputStream object, but before the first `read` call.
*/
class LazyBlockInputStream : public IBlockInputStream
{
public:
using Generator = std::function<BlockInputStreamPtr()>;
LazyBlockInputStream(const Block & header_, Generator generator_)
: header(header_), generator(std::move(generator_))
{
}
LazyBlockInputStream(const char * name_, const Block & header_, Generator generator_)
: name(name_), header(header_), generator(std::move(generator_))
{
}
String getName() const override { return name; }
Block getHeader() const override
{
return header;
}
/// We call readPrefix lazily. Suppress default behaviour.
void readPrefix() override {}
protected:
Block readImpl() override
{
if (!input)
{
input = generator();
if (!input)
return Block();
auto * p_input = dynamic_cast<IBlockInputStream *>(input.get());
if (p_input)
{
/// They could have been set before, but were not passed into the `input`.
if (progress_callback)
p_input->setProgressCallback(progress_callback);
if (process_list_elem)
p_input->setProcessListElement(process_list_elem);
}
input->readPrefix();
{
addChild(input);
if (isCancelled() && p_input)
p_input->cancel(is_killed);
}
}
return input->read();
}
private:
const char * name = "Lazy";
Block header;
Generator generator;
BlockInputStreamPtr input;
};
}

View File

@ -1,24 +0,0 @@
#pragma once
#include <DataStreams/IBlockInputStream.h>
namespace DB
{
/// Empty stream of blocks of specified structure.
class NullBlockInputStream : public IBlockInputStream
{
public:
NullBlockInputStream(const Block & header_) : header(header_) {}
Block getHeader() const override { return header; }
String getName() const override { return "Null"; }
private:
Block header;
Block readImpl() override { return {}; }
};
}

View File

@ -8,7 +8,6 @@
#include <common/logger_useful.h>
#include <DataStreams/IBlockInputStream.h>
#include <Common/setThreadName.h>
#include <Common/CurrentMetrics.h>
#include <Common/CurrentThread.h>

View File

@ -1,6 +1,6 @@
#pragma once
#include <DataStreams/IBlockInputStream.h>
#include <cstddef>
namespace DB

View File

@ -0,0 +1,23 @@
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeFactory.h>
#include <DataTypes/Serializations/SerializationDate32.h>
namespace DB
{
bool DataTypeDate32::equals(const IDataType & rhs) const
{
return typeid(rhs) == typeid(*this);
}
SerializationPtr DataTypeDate32::doGetDefaultSerialization() const
{
return std::make_shared<SerializationDate32>();
}
void registerDataTypeDate32(DataTypeFactory & factory)
{
factory.registerSimpleDataType(
"Date32", [] { return DataTypePtr(std::make_shared<DataTypeDate32>()); }, DataTypeFactory::CaseInsensitive);
}
}

View File

@ -0,0 +1,29 @@
#pragma once
#include <DataTypes/DataTypeNumberBase.h>
#include <common/DateLUT.h>
namespace DB
{
class DataTypeDate32 final : public DataTypeNumberBase<Int32>
{
public:
static constexpr auto family_name = "Date32";
TypeIndex getTypeId() const override { return TypeIndex::Date32; }
const char * getFamilyName() const override { return family_name; }
Field getDefault() const override
{
return -static_cast<Int64>(DateLUT::instance().getDayNumOffsetEpoch());
}
bool canBeUsedAsVersion() const override { return true; }
bool canBeInsideNullable() const override { return true; }
bool equals(const IDataType & rhs) const override;
protected:
SerializationPtr doGetDefaultSerialization() const override;
};
}

View File

@ -194,6 +194,7 @@ DataTypeFactory::DataTypeFactory()
registerDataTypeNumbers(*this);
registerDataTypeDecimal(*this);
registerDataTypeDate(*this);
registerDataTypeDate32(*this);
registerDataTypeDateTime(*this);
registerDataTypeString(*this);
registerDataTypeFixedString(*this);

View File

@ -69,6 +69,7 @@ private:
void registerDataTypeNumbers(DataTypeFactory & factory);
void registerDataTypeDecimal(DataTypeFactory & factory);
void registerDataTypeDate(DataTypeFactory & factory);
void registerDataTypeDate32(DataTypeFactory & factory);
void registerDataTypeDateTime(DataTypeFactory & factory);
void registerDataTypeString(DataTypeFactory & factory);
void registerDataTypeFixedString(DataTypeFactory & factory);

View File

@ -78,6 +78,8 @@ MutableColumnUniquePtr DataTypeLowCardinality::createColumnUniqueImpl(const IDat
return creator(static_cast<ColumnFixedString *>(nullptr));
else if (which.isDate())
return creator(static_cast<ColumnVector<UInt16> *>(nullptr));
else if (which.isDate32())
return creator(static_cast<ColumnVector<Int32> *>(nullptr));
else if (which.isDateTime())
return creator(static_cast<ColumnVector<UInt32> *>(nullptr));
else if (which.isUUID())

View File

@ -19,6 +19,7 @@ namespace DB
namespace ErrorCodes
{
extern const int EMPTY_DATA_PASSED;
extern const int LOGICAL_ERROR;
}
@ -27,6 +28,16 @@ DataTypePtr FieldToDataType::operator() (const Null &) const
return std::make_shared<DataTypeNullable>(std::make_shared<DataTypeNothing>());
}
DataTypePtr FieldToDataType::operator() (const NegativeInfinity &) const
{
throw Exception("It's invalid to have -inf literals in SQL", ErrorCodes::LOGICAL_ERROR);
}
DataTypePtr FieldToDataType::operator() (const PositiveInfinity &) const
{
throw Exception("It's invalid to have +inf literals in SQL", ErrorCodes::LOGICAL_ERROR);
}
DataTypePtr FieldToDataType::operator() (const UInt64 & x) const
{
if (x <= std::numeric_limits<UInt8>::max()) return std::make_shared<DataTypeUInt8>();

View File

@ -21,6 +21,8 @@ class FieldToDataType : public StaticVisitor<DataTypePtr>
{
public:
DataTypePtr operator() (const Null & x) const;
DataTypePtr operator() (const NegativeInfinity & x) const;
DataTypePtr operator() (const PositiveInfinity & x) const;
DataTypePtr operator() (const UInt64 & x) const;
DataTypePtr operator() (const UInt128 & x) const;
DataTypePtr operator() (const UInt256 & x) const;

View File

@ -322,8 +322,10 @@ struct WhichDataType
constexpr bool isEnum() const { return isEnum8() || isEnum16(); }
constexpr bool isDate() const { return idx == TypeIndex::Date; }
constexpr bool isDate32() const { return idx == TypeIndex::Date32; }
constexpr bool isDateTime() const { return idx == TypeIndex::DateTime; }
constexpr bool isDateTime64() const { return idx == TypeIndex::DateTime64; }
constexpr bool isDateOrDate32() const { return isDate() || isDate32(); }
constexpr bool isString() const { return idx == TypeIndex::String; }
constexpr bool isFixedString() const { return idx == TypeIndex::FixedString; }
@ -347,6 +349,10 @@ struct WhichDataType
template <typename T>
inline bool isDate(const T & data_type) { return WhichDataType(data_type).isDate(); }
template <typename T>
inline bool isDate32(const T & data_type) { return WhichDataType(data_type).isDate32(); }
template <typename T>
inline bool isDateOrDate32(const T & data_type) { return WhichDataType(data_type).isDateOrDate32(); }
template <typename T>
inline bool isDateTime(const T & data_type) { return WhichDataType(data_type).isDateTime(); }
template <typename T>
inline bool isDateTime64(const T & data_type) { return WhichDataType(data_type).isDateTime64(); }
@ -411,7 +417,7 @@ template <typename T>
inline bool isColumnedAsNumber(const T & data_type)
{
WhichDataType which(data_type);
return which.isInt() || which.isUInt() || which.isFloat() || which.isDate() || which.isDateTime() || which.isDateTime64() || which.isUUID();
return which.isInt() || which.isUInt() || which.isFloat() || which.isDateOrDate32() || which.isDateTime() || which.isDateTime64() || which.isUUID();
}
template <typename T>
@ -478,6 +484,7 @@ template <typename T>
class DataTypeNumber;
class DataTypeDate;
class DataTypeDate32;
class DataTypeDateTime;
class DataTypeDateTime64;
@ -487,6 +494,7 @@ template <> inline constexpr bool IsDataTypeDecimal<DataTypeDateTime64> = true;
template <typename T> constexpr bool IsDataTypeNumber<DataTypeNumber<T>> = true;
template <> inline constexpr bool IsDataTypeDateOrDateTime<DataTypeDate> = true;
template <> inline constexpr bool IsDataTypeDateOrDateTime<DataTypeDate32> = true;
template <> inline constexpr bool IsDataTypeDateOrDateTime<DataTypeDateTime> = true;
template <> inline constexpr bool IsDataTypeDateOrDateTime<DataTypeDateTime64> = true;

View File

@ -0,0 +1,78 @@
#include <DataTypes/Serializations/SerializationDate32.h>
#include <IO/ReadHelpers.h>
#include <IO/WriteHelpers.h>
#include <Columns/ColumnsNumber.h>
#include <Common/assert_cast.h>
namespace DB
{
void SerializationDate32::serializeText(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings &) const
{
writeDateText(ExtendedDayNum(assert_cast<const ColumnInt32 &>(column).getData()[row_num]), ostr);
}
void SerializationDate32::deserializeWholeText(IColumn & column, ReadBuffer & istr, const FormatSettings & settings) const
{
deserializeTextEscaped(column, istr, settings);
}
void SerializationDate32::deserializeTextEscaped(IColumn & column, ReadBuffer & istr, const FormatSettings &) const
{
ExtendedDayNum x;
readDateText(x, istr);
assert_cast<ColumnInt32 &>(column).getData().push_back(x);
}
void SerializationDate32::serializeTextEscaped(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings & settings) const
{
serializeText(column, row_num, ostr, settings);
}
void SerializationDate32::serializeTextQuoted(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings & settings) const
{
writeChar('\'', ostr);
serializeText(column, row_num, ostr, settings);
writeChar('\'', ostr);
}
void SerializationDate32::deserializeTextQuoted(IColumn & column, ReadBuffer & istr, const FormatSettings &) const
{
ExtendedDayNum x;
assertChar('\'', istr);
readDateText(x, istr);
assertChar('\'', istr);
assert_cast<ColumnInt32 &>(column).getData().push_back(x); /// It's important to do this at the end - for exception safety.
}
void SerializationDate32::serializeTextJSON(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings & settings) const
{
writeChar('"', ostr);
serializeText(column, row_num, ostr, settings);
writeChar('"', ostr);
}
void SerializationDate32::deserializeTextJSON(IColumn & column, ReadBuffer & istr, const FormatSettings &) const
{
ExtendedDayNum x;
assertChar('"', istr);
readDateText(x, istr);
assertChar('"', istr);
assert_cast<ColumnInt32 &>(column).getData().push_back(x);
}
void SerializationDate32::serializeTextCSV(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings & settings) const
{
writeChar('"', ostr);
serializeText(column, row_num, ostr, settings);
writeChar('"', ostr);
}
void SerializationDate32::deserializeTextCSV(IColumn & column, ReadBuffer & istr, const FormatSettings &) const
{
LocalDate value;
readCSV(value, istr);
assert_cast<ColumnInt32 &>(column).getData().push_back(value.getExtenedDayNum());
}
}

View File

@ -0,0 +1,21 @@
#pragma once
#include <DataTypes/Serializations/SerializationNumber.h>
namespace DB
{
class SerializationDate32 final : public SerializationNumber<Int32>
{
public:
void serializeText(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings &) const override;
void deserializeWholeText(IColumn & column, ReadBuffer & istr, const FormatSettings &) const override;
void serializeTextEscaped(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings &) const override;
void deserializeTextEscaped(IColumn & column, ReadBuffer & istr, const FormatSettings &) const override;
void serializeTextQuoted(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings &) const override;
void deserializeTextQuoted(IColumn & column, ReadBuffer & istr, const FormatSettings &) const override;
void serializeTextJSON(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings &) const override;
void deserializeTextJSON(IColumn & column, ReadBuffer & istr, const FormatSettings &) const override;
void serializeTextCSV(const IColumn & column, size_t row_num, WriteBuffer & ostr, const FormatSettings &) const override;
void deserializeTextCSV(IColumn & column, ReadBuffer & istr, const FormatSettings & settings) const override;
};
}

View File

@ -16,6 +16,7 @@ SRCS(
DataTypeCustomIPv4AndIPv6.cpp
DataTypeCustomSimpleAggregateFunction.cpp
DataTypeDate.cpp
DataTypeDate32.cpp
DataTypeDateTime.cpp
DataTypeDateTime64.cpp
DataTypeDecimalBase.cpp
@ -45,6 +46,7 @@ SRCS(
Serializations/SerializationArray.cpp
Serializations/SerializationCustomSimpleText.cpp
Serializations/SerializationDate.cpp
Serializations/SerializationDate32.cpp
Serializations/SerializationDateTime.cpp
Serializations/SerializationDateTime64.cpp
Serializations/SerializationDecimal.cpp

View File

@ -6,13 +6,13 @@
#include <Columns/ColumnVector.h>
#include <Columns/IColumn.h>
#include <Core/Names.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataTypes/DataTypesNumber.h>
#include <common/logger_useful.h>
#include "DictionaryBlockInputStreamBase.h"
#include "DictionaryStructure.h"
#include "IDictionary.h"
namespace DB
{

View File

@ -9,13 +9,14 @@
#include <Columns/ColumnArray.h>
#include <Columns/ColumnsNumber.h>
#include <Columns/ColumnNullable.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataTypes/DataTypesDecimal.h>
#include <DataTypes/DataTypeArray.h>
#include <DataTypes/DataTypeNullable.h>
#include <Core/Block.h>
#include <Dictionaries/IDictionary.h>
#include <Dictionaries/DictionaryStructure.h>
#include <DataStreams/IBlockInputStream.h>
namespace DB
{

View File

@ -2,13 +2,13 @@
#include <Core/Defines.h>
#include <Common/HashTable/HashMap.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataTypes/DataTypesDecimal.h>
#include <Functions/FunctionHelpers.h>
#include <Dictionaries/DictionaryFactory.h>
#include <Dictionaries/HierarchyDictionariesUtils.h>
namespace DB
{
namespace ErrorCodes

View File

@ -266,7 +266,7 @@ void registerDictionarySourceExecutable(DictionarySourceFactory & factory)
/// Executable dictionaries may execute arbitrary commands.
/// It's OK for dictionaries created by administrator from xml-file, but
/// maybe dangerous for dictionaries created from DDL-queries.
if (created_from_ddl)
if (created_from_ddl && context->getApplicationType() != Context::ApplicationType::LOCAL)
throw Exception(ErrorCodes::DICTIONARY_ACCESS_DENIED, "Dictionaries with executable dictionary source are not allowed to be created from DDL query");
auto context_local_copy = copyContextAndApplySettings(config_prefix, context, config);

View File

@ -283,7 +283,7 @@ void registerDictionarySourceExecutablePool(DictionarySourceFactory & factory)
/// Executable dictionaries may execute arbitrary commands.
/// It's OK for dictionaries created by administrator from xml-file, but
/// maybe dangerous for dictionaries created from DDL-queries.
if (created_from_ddl)
if (created_from_ddl && context->getApplicationType() != Context::ApplicationType::LOCAL)
throw Exception(ErrorCodes::DICTIONARY_ACCESS_DENIED, "Dictionaries with executable pool dictionary source are not allowed to be created from DDL query");
auto context_local_copy = copyContextAndApplySettings(config_prefix, context, config);

View File

@ -2,7 +2,6 @@
#include <Columns/ColumnString.h>
#include <Columns/ColumnVector.h>
#include <Columns/IColumn.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypesNumber.h>
#include <common/range.h>
@ -11,6 +10,7 @@
#include "IDictionary.h"
#include "RangeHashedDictionary.h"
namespace DB
{
/*

View File

@ -22,7 +22,9 @@ namespace
using DiskEncryptedPtr = std::shared_ptr<DiskEncrypted>;
using namespace FileEncryption;
String unhexKey(const String & hex, const String & disk_name)
constexpr Algorithm DEFAULT_ENCRYPTION_ALGORITHM = Algorithm::AES_128_CTR;
String unhexKey(const String & hex)
{
try
{
@ -30,12 +32,13 @@ namespace
}
catch (const std::exception &)
{
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Cannot read key_hex for disk {}, check for valid characters [0-9a-fA-F] and length", disk_name);
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Cannot read key_hex, check for valid characters [0-9a-fA-F] and length");
}
}
struct DiskEncryptedSettings
{
Algorithm encryption_algorithm;
String key;
DiskPtr wrapped_disk;
String path_on_wrapped_disk;
@ -43,43 +46,46 @@ namespace
DiskEncryptedSettings(
const String & disk_name, const Poco::Util::AbstractConfiguration & config, const String & config_prefix, const DisksMap & map)
{
String wrapped_disk_name = config.getString(config_prefix + ".disk", "");
if (wrapped_disk_name.empty())
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Name of the wrapped disk must not be empty. An encrypted disk is a wrapper over another disk. "
"Disk {}", disk_name);
key = config.getString(config_prefix + ".key", "");
String key_hex = config.getString(config_prefix + ".key_hex", "");
if (!key.empty() && !key_hex.empty())
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Both 'key' and 'key_hex' are specified. There should be only one. Disk {}", disk_name);
if (!key_hex.empty())
try
{
assert(key.empty());
key = unhexKey(key_hex, disk_name);
encryption_algorithm = DEFAULT_ENCRYPTION_ALGORITHM;
if (config.has(config_prefix + ".algorithm"))
parseFromString(encryption_algorithm, config.getString(config_prefix + ".algorithm"));
key = config.getString(config_prefix + ".key", "");
String key_hex = config.getString(config_prefix + ".key_hex", "");
if (!key.empty() && !key_hex.empty())
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Both 'key' and 'key_hex' are specified. There should be only one");
if (!key_hex.empty())
{
assert(key.empty());
key = unhexKey(key_hex);
}
FileEncryption::checkKeySize(encryption_algorithm, key.size());
String wrapped_disk_name = config.getString(config_prefix + ".disk", "");
if (wrapped_disk_name.empty())
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Name of the wrapped disk must not be empty. An encrypted disk is a wrapper over another disk");
auto wrapped_disk_it = map.find(wrapped_disk_name);
if (wrapped_disk_it == map.end())
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"The wrapped disk must have been announced earlier. No disk with name {}",
wrapped_disk_name);
wrapped_disk = wrapped_disk_it->second;
path_on_wrapped_disk = config.getString(config_prefix + ".path", "");
}
catch (Exception & e)
{
e.addMessage("Disk " + disk_name);
throw;
}
if (key.empty())
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Key of the encrypted disk must not be empty. Disk {}", disk_name);
if (!FileEncryption::isKeyLengthSupported(key.length()))
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Key length is not supported, supported only keys of length 16, 24, or 32 bytes. Disk {}", disk_name);
auto wrapped_disk_it = map.find(wrapped_disk_name);
if (wrapped_disk_it == map.end())
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"The wrapped disk must have been announced earlier. No disk with name {}. Disk {}",
wrapped_disk_name, disk_name);
wrapped_disk = wrapped_disk_it->second;
path_on_wrapped_disk = config.getString(config_prefix + ".path", "");
}
};
@ -123,9 +129,13 @@ ReservationPtr DiskEncrypted::reserve(UInt64 bytes)
return std::make_unique<DiskEncryptedReservation>(std::static_pointer_cast<DiskEncrypted>(shared_from_this()), std::move(reservation));
}
DiskEncrypted::DiskEncrypted(const String & name_, DiskPtr disk_, const String & key_, const String & path_)
: DiskDecorator(disk_)
, name(name_), key(key_), disk_path(path_)
DiskEncrypted::DiskEncrypted(
const String & name_,
DiskPtr wrapped_disk_,
const String & path_on_wrapped_disk_,
FileEncryption::Algorithm encryption_algorithm_,
const String & key_)
: DiskDecorator(wrapped_disk_), name(name_), disk_path(path_on_wrapped_disk_), encryption_algorithm(encryption_algorithm_), key(key_)
{
initialize();
}
@ -152,10 +162,10 @@ void DiskEncrypted::copy(const String & from_path, const std::shared_ptr<IDisk>
/// Disk type is the same, check if the key is the same too.
if (auto * to_encrypted_disk = typeid_cast<DiskEncrypted *>(to_disk.get()))
{
if (key == to_encrypted_disk->key)
if ((encryption_algorithm == to_encrypted_disk->encryption_algorithm) && (key == to_encrypted_disk->key))
{
/// Key is the same so we can simply copy the encrypted file.
delegate->copy(wrappedPath(from_path), to_encrypted_disk->delegate, wrappedPath(to_path));
delegate->copy(wrappedPath(from_path), to_encrypted_disk->delegate, to_encrypted_disk->wrappedPath(to_path));
return;
}
}
@ -178,7 +188,7 @@ std::unique_ptr<ReadBufferFromFileBase> DiskEncrypted::readFile(
InitVector iv;
iv.read(*buffer);
return std::make_unique<ReadBufferFromEncryptedFile>(buf_size, std::move(buffer), key, iv);
return std::make_unique<ReadBufferFromEncryptedFile>(buf_size, std::move(buffer), encryption_algorithm, key, iv);
}
std::unique_ptr<WriteBufferFromFileBase> DiskEncrypted::writeFile(const String & path, size_t buf_size, WriteMode mode)
@ -197,7 +207,7 @@ std::unique_ptr<WriteBufferFromFileBase> DiskEncrypted::writeFile(const String &
iv = InitVector::random();
auto buffer = delegate->writeFile(wrapped_path, buf_size, mode);
return std::make_unique<WriteBufferFromEncryptedFile>(buf_size, std::move(buffer), key, iv, old_file_size);
return std::make_unique<WriteBufferFromEncryptedFile>(buf_size, std::move(buffer), encryption_algorithm, key, iv, old_file_size);
}
@ -227,9 +237,10 @@ void DiskEncrypted::applyNewSettings(
const DisksMap & map)
{
DiskEncryptedSettings settings{name, config, config_prefix, map};
key = settings.key;
delegate = settings.wrapped_disk;
disk_path = settings.path_on_wrapped_disk;
encryption_algorithm = settings.encryption_algorithm;
key = settings.key;
initialize();
}
@ -242,7 +253,8 @@ void registerDiskEncrypted(DiskFactory & factory)
const DisksMap & map) -> DiskPtr
{
DiskEncryptedSettings settings{name, config, config_prefix, map};
return std::make_shared<DiskEncrypted>(name, settings.wrapped_disk, settings.key, settings.path_on_wrapped_disk);
return std::make_shared<DiskEncrypted>(
name, settings.wrapped_disk, settings.path_on_wrapped_disk, settings.encryption_algorithm, settings.key);
};
factory.registerDiskType("encrypted", creator);
}

View File

@ -13,6 +13,7 @@ namespace DB
{
class ReadBufferFromFileBase;
class WriteBufferFromFileBase;
namespace FileEncryption { enum class Algorithm; }
/// Encrypted disk ciphers all written files on the fly and writes the encrypted files to an underlying (normal) disk.
/// And when we read files from an encrypted disk it deciphers them automatically,
@ -20,7 +21,12 @@ class WriteBufferFromFileBase;
class DiskEncrypted : public DiskDecorator
{
public:
DiskEncrypted(const String & name_, DiskPtr disk_, const String & key_, const String & path_);
DiskEncrypted(
const String & name_,
DiskPtr wrapped_disk_,
const String & path_on_wrapped_disk_,
FileEncryption::Algorithm encryption_algorithm_,
const String & key_);
const String & getName() const override { return name; }
const String & getPath() const override { return disk_absolute_path; }
@ -219,9 +225,10 @@ private:
}
String name;
String key;
String disk_path;
String disk_absolute_path;
FileEncryption::Algorithm encryption_algorithm;
String key;
};
}

View File

@ -1,5 +1 @@
configure_file(config_formats.h.in ${ConfigIncludePath}/config_formats.h)
if (ENABLE_EXAMPLES)
add_subdirectory(examples)
endif()

View File

@ -9,7 +9,6 @@
#include <Formats/FormatSettings.h>
#include <Processors/Formats/IRowInputFormat.h>
#include <Processors/Formats/IRowOutputFormat.h>
#include <Processors/Formats/InputStreamFromInputFormat.h>
#include <Processors/Formats/OutputStreamToOutputFormat.h>
#include <Processors/Formats/Impl/ValuesBlockInputFormat.h>
#include <Processors/Formats/Impl/MySQLOutputFormat.h>
@ -209,9 +208,6 @@ BlockOutputStreamPtr FormatFactory::getOutputStreamParallelIfPossible(
WriteCallback callback,
const std::optional<FormatSettings> & _format_settings) const
{
if (context->getMySQLProtocolContext() && name != "MySQLWire")
throw Exception(ErrorCodes::UNSUPPORTED_METHOD, "MySQL protocol does not support custom output formats");
const auto & output_getter = getCreators(name).output_processor_creator;
const Settings & settings = context->getSettingsRef();
@ -316,9 +312,6 @@ OutputFormatPtr FormatFactory::getOutputFormatParallelIfPossible(
if (!output_getter)
throw Exception(ErrorCodes::FORMAT_IS_NOT_SUITABLE_FOR_OUTPUT, "Format {} is not suitable for output (with processors)", name);
if (context->getMySQLProtocolContext() && name != "MySQLWire")
throw Exception(ErrorCodes::UNSUPPORTED_METHOD, "MySQL protocol does not support custom output formats");
auto format_settings = _format_settings ? *_format_settings : getFormatSettings(context);
const Settings & settings = context->getSettingsRef();
@ -360,8 +353,11 @@ OutputFormatPtr FormatFactory::getOutputFormat(
RowOutputFormatParams params;
params.callback = std::move(callback);
auto format_settings = _format_settings
? *_format_settings : getFormatSettings(context);
auto format_settings = _format_settings ? *_format_settings : getFormatSettings(context);
/// If we're handling MySQL protocol connection right now then MySQLWire is only allowed output format.
if (format_settings.mysql_wire.sequence_id && (name != "MySQLWire"))
throw Exception(ErrorCodes::UNSUPPORTED_METHOD, "MySQL protocol does not support custom output formats");
/** TODO: Materialization is needed, because formats can use the functions `IDataType`,
* which only work with full columns.

View File

@ -132,6 +132,13 @@ struct FormatSettings
bool allow_multiple_rows_without_delimiter = false;
} protobuf;
struct
{
uint32_t client_capabilities = 0;
size_t max_packet_size = 0;
uint8_t * sequence_id = nullptr; /// Not null if it's MySQLWire output format used to handle MySQL protocol connections.
} mysql_wire;
struct
{
std::string regexp;

View File

@ -1,4 +0,0 @@
set(SRCS )
add_executable (tab_separated_streams tab_separated_streams.cpp ${SRCS})
target_link_libraries (tab_separated_streams PRIVATE clickhouse_aggregate_functions dbms)

View File

@ -1,57 +0,0 @@
#include <string>
#include <iostream>
#include <IO/ReadBufferFromFile.h>
#include <IO/WriteBufferFromFile.h>
#include <DataTypes/DataTypesNumber.h>
#include <DataTypes/DataTypeString.h>
#include <Processors/Formats/Impl/TabSeparatedRowInputFormat.h>
#include <DataStreams/copyData.h>
#include <Processors/Formats/OutputStreamToOutputFormat.h>
#include <Processors/Formats/Impl/TabSeparatedRowOutputFormat.h>
#include <Processors/Formats/InputStreamFromInputFormat.h>
using namespace DB;
int main(int, char **)
try
{
Block sample;
{
ColumnWithTypeAndName col;
col.type = std::make_shared<DataTypeUInt64>();
sample.insert(std::move(col));
}
{
ColumnWithTypeAndName col;
col.type = std::make_shared<DataTypeString>();
sample.insert(std::move(col));
}
ReadBufferFromFile in_buf("test_in");
WriteBufferFromFile out_buf("test_out");
FormatSettings format_settings;
RowInputFormatParams in_params{DEFAULT_INSERT_BLOCK_SIZE, 0, 0};
RowOutputFormatParams out_params{[](const Columns & /* columns */, size_t /* row */){}};
InputFormatPtr input_format = std::make_shared<TabSeparatedRowInputFormat>(sample, in_buf, in_params, false, false, format_settings);
BlockInputStreamPtr block_input = std::make_shared<InputStreamFromInputFormat>(std::move(input_format));
BlockOutputStreamPtr block_output = std::make_shared<OutputStreamToOutputFormat>(
std::make_shared<TabSeparatedRowOutputFormat>(out_buf, sample, false, false, out_params, format_settings));
copyData(*block_input, *block_output);
return 0;
}
catch (...)
{
std::cerr << getCurrentExceptionMessage(true) << '\n';
return 1;
}

View File

@ -42,6 +42,11 @@ struct ToYearWeekImpl
YearWeek yw = time_zone.toYearWeek(time_zone.toDayNum(t), week_mode | static_cast<UInt32>(WeekModeFlag::YEAR));
return yw.first * 100 + yw.second;
}
static inline UInt32 execute(Int32 d, UInt8 week_mode, const DateLUTImpl & time_zone)
{
YearWeek yw = time_zone.toYearWeek(ExtendedDayNum (d), week_mode | static_cast<UInt32>(WeekModeFlag::YEAR));
return yw.first * 100 + yw.second;
}
static inline UInt32 execute(UInt16 d, UInt8 week_mode, const DateLUTImpl & time_zone)
{
YearWeek yw = time_zone.toYearWeek(DayNum(d), week_mode | static_cast<UInt32>(WeekModeFlag::YEAR));
@ -65,6 +70,10 @@ struct ToStartOfWeekImpl
return time_zone.toFirstDayNumOfWeek(time_zone.toDayNum(t), week_mode);
// return time_zone.toFirstDayNumOfWeek(t, week_mode);
}
static inline UInt16 execute(Int32 d, UInt8 week_mode, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfWeek(ExtendedDayNum(d), week_mode);
}
static inline UInt16 execute(UInt16 d, UInt8 week_mode, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfWeek(ExtendedDayNum(d), week_mode);
@ -88,6 +97,11 @@ struct ToWeekImpl
YearWeek yw = time_zone.toYearWeek(time_zone.toDayNum(t), week_mode);
return yw.second;
}
static inline UInt8 execute(Int32 d, UInt8 week_mode, const DateLUTImpl & time_zone)
{
YearWeek yw = time_zone.toYearWeek(ExtendedDayNum(d), week_mode);
return yw.second;
}
static inline UInt8 execute(UInt16 d, UInt8 week_mode, const DateLUTImpl & time_zone)
{
YearWeek yw = time_zone.toYearWeek(DayNum(d), week_mode);

View File

@ -46,6 +46,7 @@ struct ZeroTransform
{
static inline UInt16 execute(Int64, const DateLUTImpl &) { return 0; }
static inline UInt16 execute(UInt32, const DateLUTImpl &) { return 0; }
static inline UInt16 execute(Int32, const DateLUTImpl &) { return 0; }
static inline UInt16 execute(UInt16, const DateLUTImpl &) { return 0; }
};
@ -61,6 +62,10 @@ struct ToDateImpl
{
return UInt16(time_zone.toDayNum(t));
}
static inline UInt16 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl &)
{
return d;
@ -69,6 +74,30 @@ struct ToDateImpl
using FactorTransform = ZeroTransform;
};
struct ToDate32Impl
{
static constexpr auto name = "toDate32";
static inline Int32 execute(Int64 t, const DateLUTImpl & time_zone)
{
return Int32(time_zone.toDayNum(t));
}
static inline Int32 execute(UInt32 t, const DateLUTImpl & time_zone)
{
return Int32(time_zone.toDayNum(t));
}
static inline Int32 execute(Int32 d, const DateLUTImpl &)
{
return d;
}
static inline Int32 execute(UInt16 d, const DateLUTImpl &)
{
return d;
}
using FactorTransform = ZeroTransform;
};
struct ToStartOfDayImpl
{
static constexpr auto name = "toStartOfDay";
@ -82,6 +111,10 @@ struct ToStartOfDayImpl
{
return time_zone.toDate(t);
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toDate(ExtendedDayNum(d));
}
static inline UInt32 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toDate(ExtendedDayNum(d));
@ -104,6 +137,10 @@ struct ToMondayImpl
//return time_zone.toFirstDayNumOfWeek(time_zone.toDayNum(t));
return time_zone.toFirstDayNumOfWeek(t);
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfWeek(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfWeek(ExtendedDayNum(d));
@ -124,6 +161,10 @@ struct ToStartOfMonthImpl
{
return time_zone.toFirstDayNumOfMonth(time_zone.toDayNum(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfMonth(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfMonth(ExtendedDayNum(d));
@ -144,6 +185,10 @@ struct ToStartOfQuarterImpl
{
return time_zone.toFirstDayNumOfQuarter(time_zone.toDayNum(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfQuarter(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfQuarter(ExtendedDayNum(d));
@ -164,6 +209,10 @@ struct ToStartOfYearImpl
{
return time_zone.toFirstDayNumOfYear(time_zone.toDayNum(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfYear(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfYear(ExtendedDayNum(d));
@ -186,7 +235,10 @@ struct ToTimeImpl
{
return time_zone.toTime(t) + 86400;
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -207,6 +259,10 @@ struct ToStartOfMinuteImpl
{
return time_zone.toStartOfMinute(t);
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -242,6 +298,10 @@ struct ToStartOfSecondImpl
{
throw Exception("Illegal type DateTime of argument for function " + std::string(name), ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -262,6 +322,10 @@ struct ToStartOfFiveMinuteImpl
{
return time_zone.toStartOfFiveMinute(t);
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -282,6 +346,10 @@ struct ToStartOfTenMinutesImpl
{
return time_zone.toStartOfTenMinutes(t);
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -302,6 +370,10 @@ struct ToStartOfFifteenMinutesImpl
{
return time_zone.toStartOfFifteenMinutes(t);
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -326,6 +398,11 @@ struct TimeSlotImpl
return t / 1800 * 1800;
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -348,6 +425,11 @@ struct ToStartOfHourImpl
return time_zone.toStartOfHour(t);
}
static inline UInt32 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt32 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -368,6 +450,10 @@ struct ToYearImpl
{
return time_zone.toYear(t);
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toYear(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toYear(ExtendedDayNum(d));
@ -388,6 +474,10 @@ struct ToQuarterImpl
{
return time_zone.toQuarter(t);
}
static inline UInt8 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toQuarter(ExtendedDayNum(d));
}
static inline UInt8 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toQuarter(ExtendedDayNum(d));
@ -408,6 +498,10 @@ struct ToMonthImpl
{
return time_zone.toMonth(t);
}
static inline UInt8 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toMonth(ExtendedDayNum(d));
}
static inline UInt8 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toMonth(ExtendedDayNum(d));
@ -428,6 +522,10 @@ struct ToDayOfMonthImpl
{
return time_zone.toDayOfMonth(t);
}
static inline UInt8 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toDayOfMonth(ExtendedDayNum(d));
}
static inline UInt8 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toDayOfMonth(ExtendedDayNum(d));
@ -448,6 +546,10 @@ struct ToDayOfWeekImpl
{
return time_zone.toDayOfWeek(t);
}
static inline UInt8 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toDayOfWeek(ExtendedDayNum(d));
}
static inline UInt8 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toDayOfWeek(ExtendedDayNum(d));
@ -468,6 +570,10 @@ struct ToDayOfYearImpl
{
return time_zone.toDayOfYear(t);
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toDayOfYear(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toDayOfYear(ExtendedDayNum(d));
@ -488,7 +594,10 @@ struct ToHourImpl
{
return time_zone.toHour(t);
}
static inline UInt8 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt8 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -511,6 +620,11 @@ struct TimezoneOffsetImpl
return time_zone.timezoneOffset(t);
}
static inline time_t execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline time_t execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -531,6 +645,10 @@ struct ToMinuteImpl
{
return time_zone.toMinute(t);
}
static inline UInt8 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt8 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -551,6 +669,10 @@ struct ToSecondImpl
{
return time_zone.toSecond(t);
}
static inline UInt8 execute(Int32, const DateLUTImpl &)
{
return dateIsNotSupported(name);
}
static inline UInt8 execute(UInt16, const DateLUTImpl &)
{
return dateIsNotSupported(name);
@ -571,6 +693,10 @@ struct ToISOYearImpl
{
return time_zone.toISOYear(time_zone.toDayNum(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toISOYear(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toISOYear(ExtendedDayNum(d));
@ -591,6 +717,10 @@ struct ToStartOfISOYearImpl
{
return time_zone.toFirstDayNumOfISOYear(time_zone.toDayNum(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfISOYear(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toFirstDayNumOfISOYear(ExtendedDayNum(d));
@ -611,6 +741,10 @@ struct ToISOWeekImpl
{
return time_zone.toISOWeek(time_zone.toDayNum(t));
}
static inline UInt8 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toISOWeek(ExtendedDayNum(d));
}
static inline UInt8 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toISOWeek(ExtendedDayNum(d));
@ -631,6 +765,10 @@ struct ToRelativeYearNumImpl
{
return time_zone.toYear(static_cast<time_t>(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toYear(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toYear(ExtendedDayNum(d));
@ -651,6 +789,10 @@ struct ToRelativeQuarterNumImpl
{
return time_zone.toRelativeQuarterNum(static_cast<time_t>(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeQuarterNum(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeQuarterNum(ExtendedDayNum(d));
@ -671,6 +813,10 @@ struct ToRelativeMonthNumImpl
{
return time_zone.toRelativeMonthNum(static_cast<time_t>(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeMonthNum(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeMonthNum(ExtendedDayNum(d));
@ -691,6 +837,10 @@ struct ToRelativeWeekNumImpl
{
return time_zone.toRelativeWeekNum(static_cast<time_t>(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeWeekNum(ExtendedDayNum(d));
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeWeekNum(ExtendedDayNum(d));
@ -711,6 +861,10 @@ struct ToRelativeDayNumImpl
{
return time_zone.toDayNum(static_cast<time_t>(t));
}
static inline UInt16 execute(Int32 d, const DateLUTImpl &)
{
return static_cast<ExtendedDayNum>(d);
}
static inline UInt16 execute(UInt16 d, const DateLUTImpl &)
{
return static_cast<DayNum>(d);
@ -732,6 +886,10 @@ struct ToRelativeHourNumImpl
{
return time_zone.toRelativeHourNum(static_cast<time_t>(t));
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeHourNum(ExtendedDayNum(d));
}
static inline UInt32 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeHourNum(ExtendedDayNum(d));
@ -752,6 +910,10 @@ struct ToRelativeMinuteNumImpl
{
return time_zone.toRelativeMinuteNum(static_cast<time_t>(t));
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeMinuteNum(ExtendedDayNum(d));
}
static inline UInt32 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toRelativeMinuteNum(ExtendedDayNum(d));
@ -772,6 +934,10 @@ struct ToRelativeSecondNumImpl
{
return t;
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.fromDayNum(ExtendedDayNum(d));
}
static inline UInt32 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.fromDayNum(ExtendedDayNum(d));
@ -792,6 +958,10 @@ struct ToYYYYMMImpl
{
return time_zone.toNumYYYYMM(t);
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toNumYYYYMM(static_cast<ExtendedDayNum>(d));
}
static inline UInt32 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toNumYYYYMM(static_cast<DayNum>(d));
@ -812,6 +982,10 @@ struct ToYYYYMMDDImpl
{
return time_zone.toNumYYYYMMDD(t);
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toNumYYYYMMDD(static_cast<ExtendedDayNum>(d));
}
static inline UInt32 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toNumYYYYMMDD(static_cast<DayNum>(d));
@ -832,6 +1006,10 @@ struct ToYYYYMMDDhhmmssImpl
{
return time_zone.toNumYYYYMMDDhhmmss(t);
}
static inline UInt64 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.toNumYYYYMMDDhhmmss(time_zone.toDate(static_cast<ExtendedDayNum>(d)));
}
static inline UInt64 execute(UInt16 d, const DateLUTImpl & time_zone)
{
return time_zone.toNumYYYYMMDDhhmmss(time_zone.toDate(static_cast<DayNum>(d)));

View File

@ -1,5 +1,6 @@
#pragma once
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
#include <Functions/CustomWeekTransforms.h>
@ -35,7 +36,7 @@ public:
{
if (arguments.size() == 1)
{
if (!isDate(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
if (!isDate(arguments[0].type) && !isDate32(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
throw Exception(
"Illegal type " + arguments[0].type->getName() + " of argument of function " + getName()
+ ". Should be a date or a date with time",
@ -43,7 +44,7 @@ public:
}
else if (arguments.size() == 2)
{
if (!isDate(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
if (!isDate(arguments[0].type) && !isDate32(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
throw Exception(
"Illegal type " + arguments[0].type->getName() + " of argument of function " + getName()
+ ". Should be a date or a date with time",
@ -59,7 +60,7 @@ public:
}
else if (arguments.size() == 3)
{
if (!isDate(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
if (!isDate(arguments[0].type) && !isDate32(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
throw Exception(
"Illegal type " + arguments[0].type->getName() + " of argument of function " + getName()
+ ". Should be a date or a date with time",
@ -105,6 +106,9 @@ public:
if (which.isDate())
return CustomWeekTransformImpl<DataTypeDate, ToDataType>::execute(
arguments, result_type, input_rows_count, Transform{});
else if (which.isDate32())
return CustomWeekTransformImpl<DataTypeDate32, ToDataType>::execute(
arguments, result_type, input_rows_count, Transform{});
else if (which.isDateTime())
return CustomWeekTransformImpl<DataTypeDateTime, ToDataType>::execute(
arguments, result_type, input_rows_count, Transform{});

View File

@ -2,6 +2,7 @@
#include <common/DateLUTImpl.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
@ -50,7 +51,11 @@ struct AddSecondsImpl
{
return t + delta;
}
static inline NO_SANITIZE_UNDEFINED Int64 execute(Int32 d, Int64 delta, const DateLUTImpl & time_zone)
{
// use default datetime64 scale
return (time_zone.fromDayNum(ExtendedDayNum(d)) + delta) * 1000;
}
static inline NO_SANITIZE_UNDEFINED UInt32 execute(UInt16 d, Int64 delta, const DateLUTImpl & time_zone)
{
return time_zone.fromDayNum(ExtendedDayNum(d)) + delta;
@ -71,7 +76,11 @@ struct AddMinutesImpl
{
return t + delta * 60;
}
static inline NO_SANITIZE_UNDEFINED Int64 execute(Int32 d, Int64 delta, const DateLUTImpl & time_zone)
{
// use default datetime64 scale
return (time_zone.fromDayNum(ExtendedDayNum(d)) + delta * 60) * 1000;
}
static inline NO_SANITIZE_UNDEFINED UInt32 execute(UInt16 d, Int64 delta, const DateLUTImpl & time_zone)
{
return time_zone.fromDayNum(ExtendedDayNum(d)) + delta * 60;
@ -91,7 +100,11 @@ struct AddHoursImpl
{
return t + delta * 3600;
}
static inline NO_SANITIZE_UNDEFINED Int64 execute(Int32 d, Int64 delta, const DateLUTImpl & time_zone)
{
// use default datetime64 scale
return (time_zone.fromDayNum(ExtendedDayNum(d)) + delta * 3600) * 1000;
}
static inline NO_SANITIZE_UNDEFINED UInt32 execute(UInt16 d, Int64 delta, const DateLUTImpl & time_zone)
{
return time_zone.fromDayNum(ExtendedDayNum(d)) + delta * 3600;
@ -117,6 +130,11 @@ struct AddDaysImpl
{
return d + delta;
}
static inline NO_SANITIZE_UNDEFINED Int32 execute(Int32 d, Int64 delta, const DateLUTImpl &)
{
return d + delta;
}
};
struct AddWeeksImpl
@ -124,17 +142,22 @@ struct AddWeeksImpl
static constexpr auto name = "addWeeks";
static inline NO_SANITIZE_UNDEFINED DecimalUtils::DecimalComponents<DateTime64>
execute(DecimalUtils::DecimalComponents<DateTime64> t, Int64 delta, const DateLUTImpl & time_zone)
execute(DecimalUtils::DecimalComponents<DateTime64> t, Int32 delta, const DateLUTImpl & time_zone)
{
return {time_zone.addWeeks(t.whole, delta), t.fractional};
}
static inline NO_SANITIZE_UNDEFINED UInt32 execute(UInt32 t, Int64 delta, const DateLUTImpl & time_zone)
static inline NO_SANITIZE_UNDEFINED UInt32 execute(UInt32 t, Int32 delta, const DateLUTImpl & time_zone)
{
return time_zone.addWeeks(t, delta);
}
static inline NO_SANITIZE_UNDEFINED UInt16 execute(UInt16 d, Int64 delta, const DateLUTImpl &)
static inline NO_SANITIZE_UNDEFINED UInt16 execute(UInt16 d, Int32 delta, const DateLUTImpl &)
{
return d + delta * 7;
}
static inline NO_SANITIZE_UNDEFINED Int32 execute(Int32 d, Int32 delta, const DateLUTImpl &)
{
return d + delta * 7;
}
@ -159,6 +182,11 @@ struct AddMonthsImpl
{
return time_zone.addMonths(ExtendedDayNum(d), delta);
}
static inline Int32 execute(Int32 d, Int64 delta, const DateLUTImpl & time_zone)
{
return time_zone.addMonths(ExtendedDayNum(d), delta);
}
};
struct AddQuartersImpl
@ -166,17 +194,22 @@ struct AddQuartersImpl
static constexpr auto name = "addQuarters";
static inline DecimalUtils::DecimalComponents<DateTime64>
execute(DecimalUtils::DecimalComponents<DateTime64> t, Int64 delta, const DateLUTImpl & time_zone)
execute(DecimalUtils::DecimalComponents<DateTime64> t, Int32 delta, const DateLUTImpl & time_zone)
{
return {time_zone.addQuarters(t.whole, delta), t.fractional};
}
static inline UInt32 execute(UInt32 t, Int64 delta, const DateLUTImpl & time_zone)
static inline UInt32 execute(UInt32 t, Int32 delta, const DateLUTImpl & time_zone)
{
return time_zone.addQuarters(t, delta);
}
static inline UInt16 execute(UInt16 d, Int64 delta, const DateLUTImpl & time_zone)
static inline UInt16 execute(UInt16 d, Int32 delta, const DateLUTImpl & time_zone)
{
return time_zone.addQuarters(ExtendedDayNum(d), delta);
}
static inline Int32 execute(Int32 d, Int32 delta, const DateLUTImpl & time_zone)
{
return time_zone.addQuarters(ExtendedDayNum(d), delta);
}
@ -201,6 +234,11 @@ struct AddYearsImpl
{
return time_zone.addYears(ExtendedDayNum(d), delta);
}
static inline Int32 execute(Int32 d, Int64 delta, const DateLUTImpl & time_zone)
{
return time_zone.addYears(ExtendedDayNum(d), delta);
}
};
template <typename Transform>
@ -342,7 +380,7 @@ template <typename FieldType> struct ResultDataTypeMap {};
template <> struct ResultDataTypeMap<UInt16> { using ResultDataType = DataTypeDate; };
template <> struct ResultDataTypeMap<Int16> { using ResultDataType = DataTypeDate; };
template <> struct ResultDataTypeMap<UInt32> { using ResultDataType = DataTypeDateTime; };
template <> struct ResultDataTypeMap<Int32> { using ResultDataType = DataTypeDateTime; };
template <> struct ResultDataTypeMap<Int32> { using ResultDataType = DataTypeDate32; };
template <> struct ResultDataTypeMap<DateTime64> { using ResultDataType = DataTypeDateTime64; };
template <> struct ResultDataTypeMap<Int64> { using ResultDataType = DataTypeDateTime64; };
}
@ -375,7 +413,7 @@ public:
if (arguments.size() == 2)
{
if (!isDate(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
if (!isDate(arguments[0].type) && !isDate32(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
throw Exception{"Illegal type " + arguments[0].type->getName() + " of first argument of function " + getName() +
". Should be a date or a date with time", ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT};
}
@ -398,6 +436,8 @@ public:
{
case TypeIndex::Date:
return resolveReturnType<DataTypeDate>(arguments);
case TypeIndex::Date32:
return resolveReturnType<DataTypeDate32>(arguments);
case TypeIndex::DateTime:
return resolveReturnType<DataTypeDateTime>(arguments);
case TypeIndex::DateTime64:
@ -437,16 +477,23 @@ public:
if constexpr (std::is_same_v<ResultDataType, DataTypeDate>)
return std::make_shared<DataTypeDate>();
else if constexpr (std::is_same_v<ResultDataType, DataTypeDate32>)
return std::make_shared<DataTypeDate32>();
else if constexpr (std::is_same_v<ResultDataType, DataTypeDateTime>)
{
return std::make_shared<DataTypeDateTime>(extractTimeZoneNameFromFunctionArguments(arguments, 2, 0));
}
else if constexpr (std::is_same_v<ResultDataType, DataTypeDateTime64>)
{
// TODO (vnemkov): what if there is an overload of Transform::execute() that returns DateTime64 from DateTime or Date ?
// Shall we use the default scale or one from optional argument ?
const auto & datetime64_type = assert_cast<const DataTypeDateTime64 &>(*arguments[0].type);
return std::make_shared<DataTypeDateTime64>(datetime64_type.getScale(), extractTimeZoneNameFromFunctionArguments(arguments, 2, 0));
if (typeid_cast<const DataTypeDateTime64 *>(arguments[0].type.get()))
{
const auto & datetime64_type = assert_cast<const DataTypeDateTime64 &>(*arguments[0].type);
return std::make_shared<DataTypeDateTime64>(datetime64_type.getScale(), extractTimeZoneNameFromFunctionArguments(arguments, 2, 0));
}
else
{
return std::make_shared<DataTypeDateTime64>(DataTypeDateTime64::default_scale, extractTimeZoneNameFromFunctionArguments(arguments, 2, 0));
}
}
else
{
@ -470,6 +517,11 @@ public:
return DateTimeAddIntervalImpl<DataTypeDate, TransformResultDataType<DataTypeDate>, Transform>::execute(
Transform{}, arguments, result_type);
}
else if (which.isDate32())
{
return DateTimeAddIntervalImpl<DataTypeDate32, TransformResultDataType<DataTypeDate32>, Transform>::execute(
Transform{}, arguments, result_type);
}
else if (which.isDateTime())
{
return DateTimeAddIntervalImpl<DataTypeDateTime, TransformResultDataType<DataTypeDateTime>, Transform>::execute(

View File

@ -1,5 +1,6 @@
#pragma once
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <Functions/IFunction.h>
#include <DataTypes/DataTypeDateTime64.h>
@ -38,7 +39,7 @@ public:
{
if (arguments.size() == 1)
{
if (!isDate(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
if (!isDateOrDate32(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
throw Exception(
"Illegal type " + arguments[0].type->getName() + " of argument of function " + getName()
+ ". Should be a date or a date with time",
@ -46,7 +47,7 @@ public:
}
else if (arguments.size() == 2)
{
if (!isDate(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
if (!isDateOrDate32(arguments[0].type) && !isDateTime(arguments[0].type) && !isDateTime64(arguments[0].type))
throw Exception(
"Illegal type " + arguments[0].type->getName() + " of argument of function " + getName()
+ ". Should be a date or a date with time",
@ -57,7 +58,7 @@ public:
"must be of type Date or DateTime. The 2nd argument (optional) must be "
"a constant string with timezone name",
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
if (isDate(arguments[0].type) && std::is_same_v<ToDataType, DataTypeDate>)
if ((isDate(arguments[0].type) || isDate32(arguments[0].type)) && (std::is_same_v<ToDataType, DataTypeDate> || std::is_same_v<ToDataType, DataTypeDate32>))
throw Exception(
"The timezone argument of function " + getName() + " is allowed only when the 1st argument has the type DateTime",
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
@ -103,6 +104,8 @@ public:
if (which.isDate())
return DateTimeTransformImpl<DataTypeDate, ToDataType, Transform>::execute(arguments, result_type, input_rows_count);
else if (which.isDate32())
return DateTimeTransformImpl<DataTypeDate32, ToDataType, Transform>::execute(arguments, result_type, input_rows_count);
else if (which.isDateTime())
return DateTimeTransformImpl<DataTypeDateTime, ToDataType, Transform>::execute(arguments, result_type, input_rows_count);
else if (which.isDateTime64())
@ -146,6 +149,12 @@ public:
== Transform::FactorTransform::execute(UInt16(right.get<UInt64>()), date_lut)
? is_monotonic : is_not_monotonic;
}
else if (checkAndGetDataType<DataTypeDate32>(&type))
{
return Transform::FactorTransform::execute(Int32(left.get<UInt64>()), date_lut)
== Transform::FactorTransform::execute(Int32(right.get<UInt64>()), date_lut)
? is_monotonic : is_not_monotonic;
}
else
{
return Transform::FactorTransform::execute(UInt32(left.get<UInt64>()), date_lut)
@ -156,4 +165,3 @@ public:
};
}

View File

@ -1081,7 +1081,7 @@ public:
const DataTypeTuple * right_tuple = checkAndGetDataType<DataTypeTuple>(arguments[1].get());
bool both_represented_by_number = arguments[0]->isValueRepresentedByNumber() && arguments[1]->isValueRepresentedByNumber();
bool has_date = left.isDate() || right.isDate();
bool has_date = left.isDateOrDate32() || right.isDateOrDate32();
if (!((both_represented_by_number && !has_date) /// Do not allow to compare date and number.
|| (left.isStringOrFixedString() || right.isStringOrFixedString()) /// Everything can be compared with string by conversion.

View File

@ -32,7 +32,7 @@ void registerFunctionsConversion(FunctionFactory & factory)
factory.registerFunction<FunctionToDate>();
/// MysQL compatibility alias.
factory.registerFunction<FunctionToDate>("DATE", FunctionFactory::CaseInsensitive);
factory.registerFunction<FunctionToDate32>();
factory.registerFunction<FunctionToDateTime>();
factory.registerFunction<FunctionToDateTime32>();
factory.registerFunction<FunctionToDateTime64>();
@ -62,6 +62,7 @@ void registerFunctionsConversion(FunctionFactory & factory)
factory.registerFunction<FunctionToFloat32OrZero>();
factory.registerFunction<FunctionToFloat64OrZero>();
factory.registerFunction<FunctionToDateOrZero>();
factory.registerFunction<FunctionToDate32OrZero>();
factory.registerFunction<FunctionToDateTimeOrZero>();
factory.registerFunction<FunctionToDateTime64OrZero>();
@ -87,6 +88,7 @@ void registerFunctionsConversion(FunctionFactory & factory)
factory.registerFunction<FunctionToFloat32OrNull>();
factory.registerFunction<FunctionToFloat64OrNull>();
factory.registerFunction<FunctionToDateOrNull>();
factory.registerFunction<FunctionToDate32OrNull>();
factory.registerFunction<FunctionToDateTimeOrNull>();
factory.registerFunction<FunctionToDateTime64OrNull>();

View File

@ -12,6 +12,7 @@
#include <DataTypes/DataTypeString.h>
#include <DataTypes/DataTypeFixedString.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
#include <DataTypes/DataTypeEnum.h>
@ -131,7 +132,7 @@ struct ConvertImpl
if (std::is_same_v<Name, NameToUnixTimestamp>)
{
if (isDate(named_from.type))
if (isDateOrDate32(named_from.type))
throw Exception("Illegal type " + named_from.type->getName() + " of first argument of function " + Name::name,
ErrorCodes::ILLEGAL_COLUMN);
}
@ -284,6 +285,10 @@ struct ConvertImpl
template <typename Name> struct ConvertImpl<DataTypeDateTime, DataTypeDate, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeDateTime, DataTypeDate, ToDateImpl> {};
/** Conversion of DateTime to Date32: throw off time component.
*/
template <typename Name> struct ConvertImpl<DataTypeDateTime, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeDateTime, DataTypeDate32, ToDate32Impl> {};
/** Conversion of Date to DateTime: adding 00:00:00 time component.
*/
@ -296,6 +301,11 @@ struct ToDateTimeImpl
return time_zone.fromDayNum(DayNum(d));
}
static inline UInt32 execute(Int32 d, const DateLUTImpl & time_zone)
{
return time_zone.fromDayNum(ExtendedDayNum(d));
}
static inline UInt32 execute(UInt32 dt, const DateLUTImpl & /*time_zone*/)
{
return dt;
@ -311,6 +321,9 @@ struct ToDateTimeImpl
template <typename Name> struct ConvertImpl<DataTypeDate, DataTypeDateTime, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeDate, DataTypeDateTime, ToDateTimeImpl> {};
template <typename Name> struct ConvertImpl<DataTypeDate32, DataTypeDateTime, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeDate32, DataTypeDateTime, ToDateTimeImpl> {};
/// Implementation of toDate function.
template <typename FromType, typename ToType>
@ -321,7 +334,7 @@ struct ToDateTransform32Or64
static inline NO_SANITIZE_UNDEFINED ToType execute(const FromType & from, const DateLUTImpl & time_zone)
{
// since converting to Date, no need in values outside of default LUT range.
return (from < 0xFFFF)
return (from < DATE_LUT_MAX_DAY_NUM)
? from
: time_zone.toDayNum(std::min(time_t(from), time_t(0xFFFFFFFF)));
}
@ -338,7 +351,7 @@ struct ToDateTransform32Or64Signed
/// The function should be monotonic (better for query optimizations), so we saturate instead of overflow.
if (from < 0)
return 0;
return (from < 0xFFFF)
return (from < DATE_LUT_MAX_DAY_NUM)
? from
: time_zone.toDayNum(std::min(time_t(from), time_t(0xFFFFFFFF)));
}
@ -357,6 +370,48 @@ struct ToDateTransform8Or16Signed
}
};
/// Implementation of toDate32 function.
template <typename FromType, typename ToType>
struct ToDate32Transform32Or64
{
static constexpr auto name = "toDate32";
static inline NO_SANITIZE_UNDEFINED ToType execute(const FromType & from, const DateLUTImpl & time_zone)
{
return (from < DATE_LUT_MAX_EXTEND_DAY_NUM)
? from
: time_zone.toDayNum(std::min(time_t(from), time_t(0xFFFFFFFF)));
}
};
template <typename FromType, typename ToType>
struct ToDate32Transform32Or64Signed
{
static constexpr auto name = "toDate32";
static inline NO_SANITIZE_UNDEFINED ToType execute(const FromType & from, const DateLUTImpl & time_zone)
{
static const Int32 daynum_min_offset = -static_cast<Int32>(DateLUT::instance().getDayNumOffsetEpoch());
if (from < daynum_min_offset)
return daynum_min_offset;
return (from < DATE_LUT_MAX_EXTEND_DAY_NUM)
? from
: time_zone.toDayNum(std::min(time_t(from), time_t(0xFFFFFFFF)));
}
};
template <typename FromType, typename ToType>
struct ToDate32Transform8Or16Signed
{
static constexpr auto name = "toDate32";
static inline NO_SANITIZE_UNDEFINED ToType execute(const FromType & from, const DateLUTImpl &)
{
return from;
}
};
/** Special case of converting Int8, Int16, (U)Int32 or (U)Int64 (and also, for convenience,
* Float32, Float64) to Date. If the number is negative, saturate it to unix epoch time. If the
* number is less than 65536, then it is treated as DayNum, and if it's greater or equals to 65536,
@ -383,6 +438,23 @@ template <typename Name> struct ConvertImpl<DataTypeFloat32, DataTypeDate, Name,
template <typename Name> struct ConvertImpl<DataTypeFloat64, DataTypeDate, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeFloat64, DataTypeDate, ToDateTransform32Or64Signed<Float64, UInt16>> {};
template <typename Name> struct ConvertImpl<DataTypeUInt32, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeUInt32, DataTypeDate32, ToDate32Transform32Or64<UInt32, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeUInt64, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeUInt64, DataTypeDate32, ToDate32Transform32Or64<UInt64, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeInt8, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeInt8, DataTypeDate32, ToDate32Transform8Or16Signed<Int8, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeInt16, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeInt16, DataTypeDate32, ToDate32Transform8Or16Signed<Int16, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeInt32, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeInt32, DataTypeDate32, ToDate32Transform32Or64Signed<Int32, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeInt64, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeInt64, DataTypeDate32, ToDate32Transform32Or64Signed<Int64, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeFloat32, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeFloat32, DataTypeDate32, ToDate32Transform32Or64Signed<Float32, Int32>> {};
template <typename Name> struct ConvertImpl<DataTypeFloat64, DataTypeDate32, Name, ConvertDefaultBehaviorTag>
: DateTimeTransformImpl<DataTypeFloat64, DataTypeDate32, ToDate32Transform32Or64Signed<Float64, Int32>> {};
template <typename FromType, typename ToType>
struct ToDateTimeTransform64
@ -598,6 +670,17 @@ struct FormatImpl<DataTypeDate>
}
};
template <>
struct FormatImpl<DataTypeDate32>
{
template <typename ReturnType = void>
static ReturnType execute(const DataTypeDate::FieldType x, WriteBuffer & wb, const DataTypeDate32 *, const DateLUTImpl *)
{
writeDateText(ExtendedDayNum(x), wb);
return ReturnType(true);
}
};
template <>
struct FormatImpl<DataTypeDateTime>
{
@ -709,6 +792,8 @@ struct ConvertImpl<FromDataType, std::enable_if_t<!std::is_same_v<FromDataType,
if constexpr (std::is_same_v<FromDataType, DataTypeDate>)
data_to.resize(size * (strlen("YYYY-MM-DD") + 1));
else if constexpr (std::is_same_v<FromDataType, DataTypeDate32>)
data_to.resize(size * (strlen("YYYY-MM-DD") + 1));
else if constexpr (std::is_same_v<FromDataType, DataTypeDateTime>)
data_to.resize(size * (strlen("YYYY-MM-DD hh:mm:ss") + 1));
else if constexpr (std::is_same_v<FromDataType, DataTypeDateTime64>)
@ -811,6 +896,14 @@ inline void parseImpl<DataTypeDate>(DataTypeDate::FieldType & x, ReadBuffer & rb
x = tmp;
}
template <>
inline void parseImpl<DataTypeDate32>(DataTypeDate32::FieldType & x, ReadBuffer & rb, const DateLUTImpl *)
{
ExtendedDayNum tmp(0);
readDateText(tmp, rb);
x = tmp;
}
// NOTE: no need of extra overload of DateTime64, since readDateTimeText64 has different signature and that case is explicitly handled in the calling code.
template <>
inline void parseImpl<DataTypeDateTime>(DataTypeDateTime::FieldType & x, ReadBuffer & rb, const DateLUTImpl * time_zone)
@ -851,6 +944,18 @@ inline bool tryParseImpl<DataTypeDate>(DataTypeDate::FieldType & x, ReadBuffer &
return true;
}
template <>
inline bool tryParseImpl<DataTypeDate32>(DataTypeDate32::FieldType & x, ReadBuffer & rb, const DateLUTImpl *)
{
ExtendedDayNum tmp(0);
if (!tryReadDateText(tmp, rb))
{
return false;
}
x = tmp;
return true;
}
template <>
inline bool tryParseImpl<DataTypeDateTime>(DataTypeDateTime::FieldType & x, ReadBuffer & rb, const DateLUTImpl * time_zone)
{
@ -1075,7 +1180,9 @@ struct ConvertThroughParsing
SerializationDecimal<typename ToDataType::FieldType>::readText(
vec_to[i], read_buffer, ToDataType::maxPrecision(), vec_to.getScale());
else
{
parseImpl<ToDataType>(vec_to[i], read_buffer, local_time_zone);
}
}
if (!isAllRead(read_buffer))
@ -1125,7 +1232,16 @@ struct ConvertThroughParsing
parsed = false;
if (!parsed)
vec_to[i] = static_cast<typename ToDataType::FieldType>(0);
{
if constexpr (std::is_same_v<ToDataType, DataTypeDate32>)
{
vec_to[i] = -static_cast<Int32>(DateLUT::instance().getDayNumOffsetEpoch());
}
else
{
vec_to[i] = static_cast<typename ToDataType::FieldType>(0);
}
}
if constexpr (exception_mode == ConvertFromStringExceptionMode::Null)
(*vec_null_map_to)[i] = !parsed;
@ -1275,6 +1391,7 @@ struct ConvertImpl<DataTypeFixedString, DataTypeString, Name, ConvertDefaultBeha
/// Declared early because used below.
struct NameToDate { static constexpr auto name = "toDate"; };
struct NameToDate32 { static constexpr auto name = "toDate32"; };
struct NameToDateTime { static constexpr auto name = "toDateTime"; };
struct NameToDateTime32 { static constexpr auto name = "toDateTime32"; };
struct NameToDateTime64 { static constexpr auto name = "toDateTime64"; };
@ -1398,6 +1515,8 @@ public:
|| std::is_same_v<Name, NameToUnixTimestamp>
// toDate(value[, timezone : String])
|| std::is_same_v<ToDataType, DataTypeDate> // TODO: shall we allow timestamp argument for toDate? DateTime knows nothing about timezones and this argument is ignored below.
// toDate(value[, timezone : String])
|| std::is_same_v<ToDataType, DataTypeDate32>
// toDateTime(value[, timezone: String])
|| std::is_same_v<ToDataType, DataTypeDateTime>
// toDateTime64(value, scale : Integer[, timezone: String])
@ -1585,7 +1704,9 @@ private:
result_column = ConvertImpl<LeftDataType, RightDataType, Name, SpecialTag>::execute(arguments, result_type, input_rows_count);
}
else
{
result_column = ConvertImpl<LeftDataType, RightDataType, Name, SpecialTag>::execute(arguments, result_type, input_rows_count);
}
return true;
};
@ -1957,7 +2078,7 @@ struct ToDateMonotonicity
static IFunction::Monotonicity get(const IDataType & type, const Field & left, const Field & right)
{
auto which = WhichDataType(type);
if (which.isDate() || which.isDateTime() || which.isDateTime64() || which.isInt8() || which.isInt16() || which.isUInt8() || which.isUInt16())
if (which.isDateOrDate32() || which.isDateTime() || which.isDateTime64() || which.isInt8() || which.isInt16() || which.isUInt8() || which.isUInt16())
return {true, true, true};
else if (
(which.isUInt() && ((left.isNull() || left.get<UInt64>() < 0xFFFF) && (right.isNull() || right.get<UInt64>() >= 0xFFFF)))
@ -1999,8 +2120,8 @@ struct ToStringMonotonicity
if (const auto * low_cardinality_type = checkAndGetDataType<DataTypeLowCardinality>(type_ptr))
type_ptr = low_cardinality_type->getDictionaryType().get();
/// `toString` function is monotonous if the argument is Date or DateTime or String, or non-negative numbers with the same number of symbols.
if (checkDataTypes<DataTypeDate, DataTypeDateTime, DataTypeString>(type_ptr))
/// `toString` function is monotonous if the argument is Date or Date32 or DateTime or String, or non-negative numbers with the same number of symbols.
if (checkDataTypes<DataTypeDate, DataTypeDate32, DataTypeDateTime, DataTypeString>(type_ptr))
return positive;
if (left.isNull() || right.isNull())
@ -2058,6 +2179,7 @@ using FunctionToInt256 = FunctionConvert<DataTypeInt256, NameToInt256, ToNumberM
using FunctionToFloat32 = FunctionConvert<DataTypeFloat32, NameToFloat32, ToNumberMonotonicity<Float32>>;
using FunctionToFloat64 = FunctionConvert<DataTypeFloat64, NameToFloat64, ToNumberMonotonicity<Float64>>;
using FunctionToDate = FunctionConvert<DataTypeDate, NameToDate, ToDateMonotonicity>;
using FunctionToDate32 = FunctionConvert<DataTypeDate32, NameToDate32, ToDateMonotonicity>;
using FunctionToDateTime = FunctionConvert<DataTypeDateTime, NameToDateTime, ToDateTimeMonotonicity>;
using FunctionToDateTime32 = FunctionConvert<DataTypeDateTime, NameToDateTime32, ToDateTimeMonotonicity>;
using FunctionToDateTime64 = FunctionConvert<DataTypeDateTime64, NameToDateTime64, UnknownMonotonicity>;
@ -2087,6 +2209,7 @@ template <> struct FunctionTo<DataTypeInt256> { using Type = FunctionToInt256; }
template <> struct FunctionTo<DataTypeFloat32> { using Type = FunctionToFloat32; };
template <> struct FunctionTo<DataTypeFloat64> { using Type = FunctionToFloat64; };
template <> struct FunctionTo<DataTypeDate> { using Type = FunctionToDate; };
template <> struct FunctionTo<DataTypeDate32> { using Type = FunctionToDate32; };
template <> struct FunctionTo<DataTypeDateTime> { using Type = FunctionToDateTime; };
template <> struct FunctionTo<DataTypeDateTime64> { using Type = FunctionToDateTime64; };
template <> struct FunctionTo<DataTypeUUID> { using Type = FunctionToUUID; };
@ -2117,6 +2240,7 @@ struct NameToInt256OrZero { static constexpr auto name = "toInt256OrZero"; };
struct NameToFloat32OrZero { static constexpr auto name = "toFloat32OrZero"; };
struct NameToFloat64OrZero { static constexpr auto name = "toFloat64OrZero"; };
struct NameToDateOrZero { static constexpr auto name = "toDateOrZero"; };
struct NameToDate32OrZero { static constexpr auto name = "toDate32OrZero"; };
struct NameToDateTimeOrZero { static constexpr auto name = "toDateTimeOrZero"; };
struct NameToDateTime64OrZero { static constexpr auto name = "toDateTime64OrZero"; };
struct NameToDecimal32OrZero { static constexpr auto name = "toDecimal32OrZero"; };
@ -2140,6 +2264,7 @@ using FunctionToInt256OrZero = FunctionConvertFromString<DataTypeInt256, NameToI
using FunctionToFloat32OrZero = FunctionConvertFromString<DataTypeFloat32, NameToFloat32OrZero, ConvertFromStringExceptionMode::Zero>;
using FunctionToFloat64OrZero = FunctionConvertFromString<DataTypeFloat64, NameToFloat64OrZero, ConvertFromStringExceptionMode::Zero>;
using FunctionToDateOrZero = FunctionConvertFromString<DataTypeDate, NameToDateOrZero, ConvertFromStringExceptionMode::Zero>;
using FunctionToDate32OrZero = FunctionConvertFromString<DataTypeDate32, NameToDate32OrZero, ConvertFromStringExceptionMode::Zero>;
using FunctionToDateTimeOrZero = FunctionConvertFromString<DataTypeDateTime, NameToDateTimeOrZero, ConvertFromStringExceptionMode::Zero>;
using FunctionToDateTime64OrZero = FunctionConvertFromString<DataTypeDateTime64, NameToDateTime64OrZero, ConvertFromStringExceptionMode::Zero>;
using FunctionToDecimal32OrZero = FunctionConvertFromString<DataTypeDecimal<Decimal32>, NameToDecimal32OrZero, ConvertFromStringExceptionMode::Zero>;
@ -2163,6 +2288,7 @@ struct NameToInt256OrNull { static constexpr auto name = "toInt256OrNull"; };
struct NameToFloat32OrNull { static constexpr auto name = "toFloat32OrNull"; };
struct NameToFloat64OrNull { static constexpr auto name = "toFloat64OrNull"; };
struct NameToDateOrNull { static constexpr auto name = "toDateOrNull"; };
struct NameToDate32OrNull { static constexpr auto name = "toDate32OrNull"; };
struct NameToDateTimeOrNull { static constexpr auto name = "toDateTimeOrNull"; };
struct NameToDateTime64OrNull { static constexpr auto name = "toDateTime64OrNull"; };
struct NameToDecimal32OrNull { static constexpr auto name = "toDecimal32OrNull"; };
@ -2186,6 +2312,7 @@ using FunctionToInt256OrNull = FunctionConvertFromString<DataTypeInt256, NameToI
using FunctionToFloat32OrNull = FunctionConvertFromString<DataTypeFloat32, NameToFloat32OrNull, ConvertFromStringExceptionMode::Null>;
using FunctionToFloat64OrNull = FunctionConvertFromString<DataTypeFloat64, NameToFloat64OrNull, ConvertFromStringExceptionMode::Null>;
using FunctionToDateOrNull = FunctionConvertFromString<DataTypeDate, NameToDateOrNull, ConvertFromStringExceptionMode::Null>;
using FunctionToDate32OrNull = FunctionConvertFromString<DataTypeDate32, NameToDate32OrNull, ConvertFromStringExceptionMode::Null>;
using FunctionToDateTimeOrNull = FunctionConvertFromString<DataTypeDateTime, NameToDateTimeOrNull, ConvertFromStringExceptionMode::Null>;
using FunctionToDateTime64OrNull = FunctionConvertFromString<DataTypeDateTime64, NameToDateTime64OrNull, ConvertFromStringExceptionMode::Null>;
using FunctionToDecimal32OrNull = FunctionConvertFromString<DataTypeDecimal<Decimal32>, NameToDecimal32OrNull, ConvertFromStringExceptionMode::Null>;
@ -2475,7 +2602,7 @@ private:
UInt32 scale = to_type->getScale();
WhichDataType which(type_index);
bool ok = which.isNativeInt() || which.isNativeUInt() || which.isDecimal() || which.isFloat() || which.isDate() || which.isDateTime() || which.isDateTime64()
bool ok = which.isNativeInt() || which.isNativeUInt() || which.isDecimal() || which.isFloat() || which.isDateOrDate32() || which.isDateTime() || which.isDateTime64()
|| which.isStringOrFixedString();
if (!ok)
{
@ -3137,6 +3264,7 @@ private:
std::is_same_v<ToDataType, DataTypeFloat32> ||
std::is_same_v<ToDataType, DataTypeFloat64> ||
std::is_same_v<ToDataType, DataTypeDate> ||
std::is_same_v<ToDataType, DataTypeDate32> ||
std::is_same_v<ToDataType, DataTypeDateTime> ||
std::is_same_v<ToDataType, DataTypeUUID>)
{
@ -3236,6 +3364,8 @@ public:
return monotonicityForType(type);
if (const auto * type = checkAndGetDataType<DataTypeDate>(to_type))
return monotonicityForType(type);
if (const auto * type = checkAndGetDataType<DataTypeDate32>(to_type))
return monotonicityForType(type);
if (const auto * type = checkAndGetDataType<DataTypeDateTime>(to_type))
return monotonicityForType(type);
if (const auto * type = checkAndGetDataType<DataTypeString>(to_type))

View File

@ -683,6 +683,8 @@ public:
return executeType<Int64>(arguments);
else if (which.isDate())
return executeType<UInt16>(arguments);
else if (which.isDate32())
return executeType<Int32>(arguments);
else if (which.isDateTime())
return executeType<UInt32>(arguments);
else if (which.isDecimal32())
@ -986,6 +988,7 @@ private:
else if (which.isEnum8()) executeIntType<Int8, first>(icolumn, vec_to);
else if (which.isEnum16()) executeIntType<Int16, first>(icolumn, vec_to);
else if (which.isDate()) executeIntType<UInt16, first>(icolumn, vec_to);
else if (which.isDate32()) executeIntType<Int32, first>(icolumn, vec_to);
else if (which.isDateTime()) executeIntType<UInt32, first>(icolumn, vec_to);
/// TODO: executeIntType() for Decimal32/64 leads to incompatible result
else if (which.isDecimal32()) executeBigIntType<Decimal32, first>(icolumn, vec_to);

View File

@ -6,6 +6,7 @@
#include <DataTypes/DataTypesNumber.h>
#include <DataTypes/DataTypesDecimal.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
#include <DataTypes/DataTypeNullable.h>
@ -407,6 +408,9 @@ ColumnPtr FunctionArrayIntersect::executeImpl(const ColumnsWithTypeAndName & arg
using DateMap = ClearableHashMapWithStackMemory<DataTypeDate::FieldType,
size_t, DefaultHash<DataTypeDate::FieldType>, INITIAL_SIZE_DEGREE>;
using Date32Map = ClearableHashMapWithStackMemory<DataTypeDate32::FieldType,
size_t, DefaultHash<DataTypeDate32::FieldType>, INITIAL_SIZE_DEGREE>;
using DateTimeMap = ClearableHashMapWithStackMemory<
DataTypeDateTime::FieldType, size_t,
DefaultHash<DataTypeDateTime::FieldType>, INITIAL_SIZE_DEGREE>;
@ -421,6 +425,8 @@ ColumnPtr FunctionArrayIntersect::executeImpl(const ColumnsWithTypeAndName & arg
if (which.isDate())
result_column = execute<DateMap, ColumnVector<DataTypeDate::FieldType>, true>(arrays, std::move(column));
else if (which.isDate32())
result_column = execute<Date32Map, ColumnVector<DataTypeDate32::FieldType>, true>(arrays, std::move(column));
else if (which.isDateTime())
result_column = execute<DateTimeMap, ColumnVector<DataTypeDateTime::FieldType>, true>(arrays, std::move(column));
else if (which.isString())

View File

@ -5,6 +5,7 @@
#include <Columns/ColumnNullable.h>
#include <Common/assert_cast.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
#include <DataTypes/DataTypesNumber.h>
@ -115,6 +116,8 @@ private:
f(Float64());
else if (which.isDate())
f(DataTypeDate::FieldType());
else if (which.isDate32())
f(DataTypeDate::FieldType());
else if (which.isDateTime())
f(DataTypeDateTime::FieldType());
else

View File

@ -1,6 +1,7 @@
#include <common/DateLUTImpl.h>
#include <Columns/ColumnsNumber.h>
#include <DataTypes/DataTypeDate.h>
#include <DataTypes/DataTypeDate32.h>
#include <DataTypes/DataTypeDateTime.h>
#include <DataTypes/DataTypeDateTime64.h>
#include <DataTypes/DataTypeInterval.h>
@ -39,6 +40,11 @@ namespace
return time_zone.toStartOfYearInterval(ExtendedDayNum(d), years);
}
static UInt16 execute(Int32 d, UInt64 years, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfYearInterval(ExtendedDayNum(d), years);
}
static UInt16 execute(UInt32 t, UInt64 years, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfYearInterval(time_zone.toDayNum(t), years);
@ -60,6 +66,11 @@ namespace
return time_zone.toStartOfQuarterInterval(ExtendedDayNum(d), quarters);
}
static UInt16 execute(Int32 d, UInt64 quarters, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfQuarterInterval(ExtendedDayNum(d), quarters);
}
static UInt16 execute(UInt32 t, UInt64 quarters, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfQuarterInterval(time_zone.toDayNum(t), quarters);
@ -81,6 +92,11 @@ namespace
return time_zone.toStartOfMonthInterval(ExtendedDayNum(d), months);
}
static UInt16 execute(Int32 d, UInt64 months, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfMonthInterval(ExtendedDayNum(d), months);
}
static UInt16 execute(UInt32 t, UInt64 months, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfMonthInterval(time_zone.toDayNum(t), months);
@ -102,6 +118,11 @@ namespace
return time_zone.toStartOfWeekInterval(ExtendedDayNum(d), weeks);
}
static UInt16 execute(Int32 d, UInt64 weeks, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfWeekInterval(ExtendedDayNum(d), weeks);
}
static UInt16 execute(UInt32 t, UInt64 weeks, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfWeekInterval(time_zone.toDayNum(t), weeks);
@ -123,6 +144,11 @@ namespace
return time_zone.toStartOfDayInterval(ExtendedDayNum(d), days);
}
static UInt32 execute(Int32 d, UInt64 days, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfDayInterval(ExtendedDayNum(d), days);
}
static UInt32 execute(UInt32 t, UInt64 days, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfDayInterval(time_zone.toDayNum(t), days);
@ -140,6 +166,7 @@ namespace
static constexpr auto name = function_name;
static UInt32 execute(UInt16, UInt64, const DateLUTImpl &) { return dateIsNotSupported(function_name); }
static UInt32 execute(Int32, UInt64, const DateLUTImpl &) { return dateIsNotSupported(function_name); }
static UInt32 execute(UInt32 t, UInt64 hours, const DateLUTImpl & time_zone) { return time_zone.toStartOfHourInterval(t, hours); }
static UInt32 execute(Int64 t, UInt64 hours, const DateLUTImpl & time_zone) { return time_zone.toStartOfHourInterval(t, hours); }
};
@ -151,6 +178,8 @@ namespace
static UInt32 execute(UInt16, UInt64, const DateLUTImpl &) { return dateIsNotSupported(function_name); }
static UInt32 execute(Int32, UInt64, const DateLUTImpl &) { return dateIsNotSupported(function_name); }
static UInt32 execute(UInt32 t, UInt64 minutes, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfMinuteInterval(t, minutes);
@ -169,6 +198,8 @@ namespace
static UInt32 execute(UInt16, UInt64, const DateLUTImpl &) { return dateIsNotSupported(function_name); }
static UInt32 execute(Int32, UInt64, const DateLUTImpl &) { return dateIsNotSupported(function_name); }
static UInt32 execute(UInt32 t, UInt64 seconds, const DateLUTImpl & time_zone)
{
return time_zone.toStartOfSecondInterval(t, seconds);
@ -299,6 +330,12 @@ private:
if (time_column_vec)
return dispatchForIntervalColumn(assert_cast<const DataTypeDate&>(from_datatype), *time_column_vec, interval_column, time_zone);
}
if (which_type.isDate32())
{
const auto * time_column_vec = checkAndGetColumn<ColumnInt32>(time_column.column.get());
if (time_column_vec)
return dispatchForIntervalColumn(assert_cast<const DataTypeDate32&>(from_datatype), *time_column_vec, interval_column, time_zone);
}
if (which_type.isDateTime64())
{
const auto * time_column_vec = checkAndGetColumn<DataTypeDateTime64::ColumnType>(time_column.column.get());

View File

@ -6,6 +6,7 @@
#include <IO/WriteBuffer.h>
#include <IO/WriteHelpers.h>
#include <boost/algorithm/string/predicate.hpp>
#include <cassert>
#include <random>
@ -15,6 +16,7 @@ namespace DB
namespace ErrorCodes
{
extern const int BAD_ARGUMENTS;
extern const int DATA_ENCRYPTION_ERROR;
}
@ -23,6 +25,41 @@ namespace FileEncryption
namespace
{
const EVP_CIPHER * getCipher(Algorithm algorithm)
{
switch (algorithm)
{
case Algorithm::AES_128_CTR: return EVP_aes_128_ctr();
case Algorithm::AES_192_CTR: return EVP_aes_192_ctr();
case Algorithm::AES_256_CTR: return EVP_aes_256_ctr();
}
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Encryption algorithm {} is not supported, specify one of the following: aes_128_ctr, aes_192_ctr, aes_256_ctr",
std::to_string(static_cast<int>(algorithm)));
}
void checkKeySize(const EVP_CIPHER * evp_cipher, size_t key_size)
{
if (!key_size)
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Encryption key must not be empty");
size_t expected_key_size = static_cast<size_t>(EVP_CIPHER_key_length(evp_cipher));
if (key_size != expected_key_size)
throw Exception(
ErrorCodes::BAD_ARGUMENTS, "Got an encryption key with unexpected size {}, the size should be {}", key_size, expected_key_size);
}
void checkInitVectorSize(const EVP_CIPHER * evp_cipher)
{
size_t expected_iv_length = static_cast<size_t>(EVP_CIPHER_iv_length(evp_cipher));
if (InitVector::kSize != expected_iv_length)
throw Exception(
ErrorCodes::DATA_ENCRYPTION_ERROR,
"Got an initialization vector with unexpected size {}, the size should be {}",
InitVector::kSize,
expected_iv_length);
}
constexpr const size_t kBlockSize = 16;
size_t blockOffset(size_t pos) { return pos % kBlockSize; }
@ -145,6 +182,39 @@ namespace
}
}
String toString(Algorithm algorithm)
{
switch (algorithm)
{
case Algorithm::AES_128_CTR: return "aes_128_ctr";
case Algorithm::AES_192_CTR: return "aes_192_ctr";
case Algorithm::AES_256_CTR: return "aes_256_ctr";
}
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Encryption algorithm {} is not supported, specify one of the following: aes_128_ctr, aes_192_ctr, aes_256_ctr",
std::to_string(static_cast<int>(algorithm)));
}
void parseFromString(Algorithm & algorithm, const String & str)
{
if (boost::iequals(str, "aes_128_ctr"))
algorithm = Algorithm::AES_128_CTR;
else if (boost::iequals(str, "aes_192_ctr"))
algorithm = Algorithm::AES_192_CTR;
else if (boost::iequals(str, "aes_256_ctr"))
algorithm = Algorithm::AES_256_CTR;
else
throw Exception(
ErrorCodes::BAD_ARGUMENTS,
"Encryption algorithm '{}' is not supported, specify one of the following: aes_128_ctr, aes_192_ctr, aes_256_ctr",
str);
}
void checkKeySize(Algorithm algorithm, size_t key_size) { checkKeySize(getCipher(algorithm), key_size); }
String InitVector::toString() const
{
static_assert(sizeof(counter) == InitVector::kSize);
@ -156,7 +226,7 @@ String InitVector::toString() const
InitVector InitVector::fromString(const String & str)
{
if (str.length() != InitVector::kSize)
throw Exception(ErrorCodes::DATA_ENCRYPTION_ERROR, "Expected iv with size {}, got iv with size {}", InitVector::kSize, str.length());
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Expected iv with size {}, got iv with size {}", InitVector::kSize, str.length());
ReadBufferFromMemory in{str.data(), str.length()};
UInt128 counter;
readBinaryBigEndian(counter, in);
@ -185,26 +255,13 @@ InitVector InitVector::random()
}
Encryptor::Encryptor(const String & key_, const InitVector & iv_)
Encryptor::Encryptor(Algorithm algorithm_, const String & key_, const InitVector & iv_)
: key(key_)
, init_vector(iv_)
, evp_cipher(getCipher(algorithm_))
{
if (key_.length() == 16)
evp_cipher = EVP_aes_128_ctr();
else if (key_.length() == 24)
evp_cipher = EVP_aes_192_ctr();
else if (key_.length() == 32)
evp_cipher = EVP_aes_256_ctr();
else
throw Exception(ErrorCodes::DATA_ENCRYPTION_ERROR, "Key length {} is not supported, supported only keys of length 128, 192, or 256 bits", key_.length());
size_t cipher_key_length = static_cast<size_t>(EVP_CIPHER_key_length(evp_cipher));
if (cipher_key_length != key_.length())
throw Exception(ErrorCodes::DATA_ENCRYPTION_ERROR, "Got unexpected key length from cipher: {} != {}", cipher_key_length, key_.length());
size_t cipher_iv_length = static_cast<size_t>(EVP_CIPHER_iv_length(evp_cipher));
if (cipher_iv_length != InitVector::kSize)
throw Exception(ErrorCodes::DATA_ENCRYPTION_ERROR, "Got unexpected init vector's length from cipher: {} != {}", cipher_iv_length, InitVector::kSize);
checkKeySize(evp_cipher, key.size());
checkInitVectorSize(evp_cipher);
}
void Encryptor::encrypt(const char * data, size_t size, WriteBuffer & out)

View File

@ -16,6 +16,24 @@ class WriteBuffer;
namespace FileEncryption
{
/// Encryption algorithm.
/// We chose to use CTR cipther algorithms because they have the following features which are important for us:
/// - No right padding, so we can append encrypted files without deciphering;
/// - One byte is always ciphered as one byte, so we get random access to encrypted files easily.
enum class Algorithm
{
AES_128_CTR, /// Size of key is 16 bytes.
AES_192_CTR, /// Size of key is 24 bytes.
AES_256_CTR, /// Size of key is 32 bytes.
};
String toString(Algorithm algorithm);
void parseFromString(Algorithm & algorithm, const String & str);
/// Throws an exception if a specified key size doesn't correspond a specified encryption algorithm.
void checkKeySize(Algorithm algorithm, size_t key_size);
/// Initialization vector. Its size is always 16 bytes.
class InitVector
{
@ -51,17 +69,12 @@ private:
UInt128 counter = 0;
};
/// Encrypts or decrypts data.
class Encryptor
{
public:
/// The `key` should have length 128 or 192 or 256.
/// According to the key's length aes_128_ctr or aes_192_ctr or aes_256_ctr will be used for encryption.
/// We chose to use CTR cipther algorithms because they have the following features which are important for us:
/// - No right padding, so we can append encrypted files without deciphering;
/// - One byte is always ciphered as one byte, so we get random access to encrypted files easily.
Encryptor(const String & key_, const InitVector & iv_);
/// The `key` should have size 16 or 24 or 32 bytes depending on which `algorithm` is specified.
Encryptor(Algorithm algorithm_, const String & key_, const InitVector & iv_);
/// Sets the current position in the data stream from the very beginning of data.
/// It affects how the data will be encrypted or decrypted because
@ -82,17 +95,12 @@ public:
private:
const String key;
const InitVector init_vector;
const EVP_CIPHER * evp_cipher;
const EVP_CIPHER * const evp_cipher;
/// The current position in the data stream from the very beginning of data.
size_t offset = 0;
};
/// Checks whether a passed key length is supported, i.e.
/// whether its length is 128 or 192 or 256 bits (16 or 24 or 32 bytes).
bool isKeyLengthSupported(size_t key_length);
}
}

View File

@ -2,6 +2,7 @@
#include <atomic>
#include <cstddef>
#include <functional>
#include <common/types.h>
#include <Core/Defines.h>
@ -120,4 +121,12 @@ struct Progress
}
};
/** Callback to track the progress of the query.
* Used in IBlockInputStream and Context.
* The function takes the number of rows in the last block, the number of bytes in the last block.
* Note that the callback can be called from different threads.
*/
using ProgressCallback = std::function<void(const Progress & progress)>;
}

View File

@ -14,12 +14,13 @@ using InitVector = FileEncryption::InitVector;
ReadBufferFromEncryptedFile::ReadBufferFromEncryptedFile(
size_t buffer_size_,
std::unique_ptr<ReadBufferFromFileBase> in_,
FileEncryption::Algorithm encryption_algorithm_,
const String & key_,
const InitVector & init_vector_)
: ReadBufferFromFileBase(buffer_size_, nullptr, 0)
, in(std::move(in_))
, encrypted_buffer(buffer_size_)
, encryptor(key_, init_vector_)
, encryptor(encryption_algorithm_, key_, init_vector_)
{
/// We should start reading from `in` at the offset == InitVector::kSize.
need_seek = true;

View File

@ -19,6 +19,7 @@ public:
ReadBufferFromEncryptedFile(
size_t buffer_size_,
std::unique_ptr<ReadBufferFromFileBase> in_,
FileEncryption::Algorithm encryption_algorithm_,
const String & key_,
const FileEncryption::InitVector & init_vector_);

View File

@ -632,6 +632,22 @@ inline ReturnType readDateTextImpl(DayNum & date, ReadBuffer & buf)
return ReturnType(true);
}
template <typename ReturnType = void>
inline ReturnType readDateTextImpl(ExtendedDayNum & date, ReadBuffer & buf)
{
static constexpr bool throw_exception = std::is_same_v<ReturnType, void>;
LocalDate local_date;
if constexpr (throw_exception)
readDateTextImpl<ReturnType>(local_date, buf);
else if (!readDateTextImpl<ReturnType>(local_date, buf))
return false;
/// When the parameter is out of rule or out of range, Date32 uses 1925-01-01 as the default value (-DateLUT::instance().getDayNumOffsetEpoch(), -16436) and Date uses 1970-01-01.
date = DateLUT::instance().makeDayNum(local_date.year(), local_date.month(), local_date.day(), -static_cast<Int32>(DateLUT::instance().getDayNumOffsetEpoch()));
return ReturnType(true);
}
inline void readDateText(LocalDate & date, ReadBuffer & buf)
{
@ -643,6 +659,11 @@ inline void readDateText(DayNum & date, ReadBuffer & buf)
readDateTextImpl<void>(date, buf);
}
inline void readDateText(ExtendedDayNum & date, ReadBuffer & buf)
{
readDateTextImpl<void>(date, buf);
}
inline bool tryReadDateText(LocalDate & date, ReadBuffer & buf)
{
return readDateTextImpl<bool>(date, buf);
@ -653,6 +674,11 @@ inline bool tryReadDateText(DayNum & date, ReadBuffer & buf)
return readDateTextImpl<bool>(date, buf);
}
inline bool tryReadDateText(ExtendedDayNum & date, ReadBuffer & buf)
{
return readDateTextImpl<bool>(date, buf);
}
template <typename ReturnType = void>
inline ReturnType readUUIDTextImpl(UUID & uuid, ReadBuffer & buf)
{

View File

@ -11,6 +11,7 @@ using InitVector = FileEncryption::InitVector;
WriteBufferFromEncryptedFile::WriteBufferFromEncryptedFile(
size_t buffer_size_,
std::unique_ptr<WriteBufferFromFileBase> out_,
FileEncryption::Algorithm encryption_algorithm_,
const String & key_,
const InitVector & init_vector_,
size_t old_file_size)
@ -18,7 +19,7 @@ WriteBufferFromEncryptedFile::WriteBufferFromEncryptedFile(
, out(std::move(out_))
, iv(init_vector_)
, flush_iv(!old_file_size)
, encryptor(key_, init_vector_)
, encryptor(encryption_algorithm_, key_, init_vector_)
{
encryptor.setOffset(old_file_size);
}

View File

@ -20,6 +20,7 @@ public:
WriteBufferFromEncryptedFile(
size_t buffer_size_,
std::unique_ptr<WriteBufferFromFileBase> out_,
FileEncryption::Algorithm encryption_algorithm_,
const String & key_,
const FileEncryption::InitVector & init_vector_,
size_t old_file_size = 0);

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