Merge branch 'master' into update-h3

This commit is contained in:
Alexey Milovidov 2021-07-10 11:52:39 +03:00
commit ee4b261857
62 changed files with 941 additions and 148 deletions

View File

@ -2,11 +2,11 @@
# NOTE: has nothing common with DBMS_TCP_PROTOCOL_VERSION,
# only DBMS_TCP_PROTOCOL_VERSION should be incremented on protocol changes.
SET(VERSION_REVISION 54453)
SET(VERSION_REVISION 54454)
SET(VERSION_MAJOR 21)
SET(VERSION_MINOR 8)
SET(VERSION_MINOR 9)
SET(VERSION_PATCH 1)
SET(VERSION_GITHASH fb895056568e26200629c7d19626e92d2dedc70d)
SET(VERSION_DESCRIBE v21.8.1.1-prestable)
SET(VERSION_STRING 21.8.1.1)
SET(VERSION_GITHASH f48c5af90c2ad51955d1ee3b6b05d006b03e4238)
SET(VERSION_DESCRIBE v21.9.1.1-prestable)
SET(VERSION_STRING 21.9.1.1)
# end of autochange

4
debian/changelog vendored
View File

@ -1,5 +1,5 @@
clickhouse (21.8.1.1) unstable; urgency=low
clickhouse (21.9.1.1) unstable; urgency=low
* Modified source code
-- clickhouse-release <clickhouse-release@yandex-team.ru> Mon, 28 Jun 2021 00:50:15 +0300
-- clickhouse-release <clickhouse-release@yandex-team.ru> Sat, 10 Jul 2021 08:22:49 +0300

View File

@ -1,7 +1,7 @@
FROM ubuntu:18.04
ARG repository="deb https://repo.clickhouse.tech/deb/stable/ main/"
ARG version=21.8.1.*
ARG version=21.9.1.*
RUN apt-get update \
&& apt-get install --yes --no-install-recommends \

View File

@ -1,7 +1,7 @@
FROM ubuntu:20.04
ARG repository="deb https://repo.clickhouse.tech/deb/stable/ main/"
ARG version=21.8.1.*
ARG version=21.9.1.*
ARG gosu_ver=1.10
# set non-empty deb_location_url url to create a docker image

View File

@ -1,7 +1,7 @@
FROM ubuntu:18.04
ARG repository="deb https://repo.clickhouse.tech/deb/stable/ main/"
ARG version=21.8.1.*
ARG version=21.9.1.*
RUN apt-get update && \
apt-get install -y apt-transport-https dirmngr && \

View File

@ -22,6 +22,23 @@ Some settings specified in the main configuration file can be overridden in othe
The config can also define “substitutions”. If an element has the `incl` attribute, the corresponding substitution from the file will be used as the value. By default, the path to the file with substitutions is `/etc/metrika.xml`. This can be changed in the [include_from](../operations/server-configuration-parameters/settings.md#server_configuration_parameters-include_from) element in the server config. The substitution values are specified in `/yandex/substitution_name` elements in this file. If a substitution specified in `incl` does not exist, it is recorded in the log. To prevent ClickHouse from logging missing substitutions, specify the `optional="true"` attribute (for example, settings for [macros](../operations/server-configuration-parameters/settings.md)).
If you want to replace an entire element with a substitution use `include` as element name.
XML substitution example:
```xml
<yandex>
<!-- Appends XML subtree found at `/profiles-in-zookeeper` ZK path to `<profiles>` element. -->
<profiles from_zk="/profiles-in-zookeeper" />
<users>
<!-- Replaces `include` element with the subtree found at `/users-in-zookeeper` ZK path. -->
<include from_zk="/users-in-zookeeper" />
<include from_zk="/other-users-in-zookeeper" />
</users>
</yandex>
```
Substitutions can also be performed from ZooKeeper. To do this, specify the attribute `from_zk = "/path/to/node"`. The element value is replaced with the contents of the node at `/path/to/node` in ZooKeeper. You can also put an entire XML subtree on the ZooKeeper node and it will be fully inserted into the source element.
## User Settings {#user-settings}
@ -32,6 +49,8 @@ Users configuration can be splitted into separate files similar to `config.xml`
Directory name is defined as `users_config` setting without `.xml` postfix concatenated with `.d`.
Directory `users.d` is used by default, as `users_config` defaults to `users.xml`.
Note that configuration files are first merged taking into account [Override](#override) settings and includes are processed after that.
## XML example {#example}
For example, you can have separate config file for each user like this:

View File

@ -12,7 +12,7 @@ For information on connecting and configuring external dictionaries, see [Extern
## dictGet, dictGetOrDefault, dictGetOrNull {#dictget}
Retrieves values from an external dictionary.
Retrieves values from an external dictionary.
``` sql
dictGet('dict_name', attr_names, id_expr)
@ -24,7 +24,7 @@ dictGetOrNull('dict_name', attr_name, id_expr)
- `dict_name` — Name of the dictionary. [String literal](../../sql-reference/syntax.md#syntax-string-literal).
- `attr_names` — Name of the column of the dictionary, [String literal](../../sql-reference/syntax.md#syntax-string-literal), or tuple of column names, [Tuple](../../sql-reference/data-types/tuple.md)([String literal](../../sql-reference/syntax.md#syntax-string-literal)).
- `id_expr` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning a [UInt64](../../sql-reference/data-types/int-uint.md) or [Tuple](../../sql-reference/data-types/tuple.md)-type value depending on the dictionary configuration.
- `id_expr` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning dictionary key-type value or [Tuple](../../sql-reference/data-types/tuple.md)-type value depending on the dictionary configuration.
- `default_value_expr` — Values returned if the dictionary does not contain a row with the `id_expr` key. [Expression](../../sql-reference/syntax.md#syntax-expressions) or [Tuple](../../sql-reference/data-types/tuple.md)([Expression](../../sql-reference/syntax.md#syntax-expressions)), returning the value (or values) in the data types configured for the `attr_names` attribute.
**Returned value**
@ -138,7 +138,7 @@ Configure the external dictionary:
<name>c2</name>
<type>String</type>
<null_value></null_value>
</attribute>
</attribute>
</structure>
<lifetime>0</lifetime>
</dictionary>
@ -237,7 +237,7 @@ dictHas('dict_name', id_expr)
**Arguments**
- `dict_name` — Name of the dictionary. [String literal](../../sql-reference/syntax.md#syntax-string-literal).
- `id_expr` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning a [UInt64](../../sql-reference/data-types/int-uint.md) or [Tuple](../../sql-reference/data-types/tuple.md)-type value depending on the dictionary configuration.
- `id_expr` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning dictionary key-type value or [Tuple](../../sql-reference/data-types/tuple.md)-type value depending on the dictionary configuration.
**Returned value**
@ -292,16 +292,16 @@ Type: `UInt8`.
Returns first-level children as an array of indexes. It is the inverse transformation for [dictGetHierarchy](#dictgethierarchy).
**Syntax**
**Syntax**
``` sql
dictGetChildren(dict_name, key)
```
**Arguments**
**Arguments**
- `dict_name` — Name of the dictionary. [String literal](../../sql-reference/syntax.md#syntax-string-literal).
- `key` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning a [UInt64](../../sql-reference/data-types/int-uint.md)-type value.
- `dict_name` — Name of the dictionary. [String literal](../../sql-reference/syntax.md#syntax-string-literal).
- `key` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning a [UInt64](../../sql-reference/data-types/int-uint.md)-type value.
**Returned values**
@ -339,7 +339,7 @@ SELECT dictGetChildren('hierarchy_flat_dictionary', number) FROM system.numbers
## dictGetDescendant {#dictgetdescendant}
Returns all descendants as if [dictGetChildren](#dictgetchildren) function was applied `level` times recursively.
Returns all descendants as if [dictGetChildren](#dictgetchildren) function was applied `level` times recursively.
**Syntax**
@ -347,9 +347,9 @@ Returns all descendants as if [dictGetChildren](#dictgetchildren) function was a
dictGetDescendants(dict_name, key, level)
```
**Arguments**
**Arguments**
- `dict_name` — Name of the dictionary. [String literal](../../sql-reference/syntax.md#syntax-string-literal).
- `dict_name` — Name of the dictionary. [String literal](../../sql-reference/syntax.md#syntax-string-literal).
- `key` — Key value. [Expression](../../sql-reference/syntax.md#syntax-expressions) returning a [UInt64](../../sql-reference/data-types/int-uint.md)-type value.
- `level` — Hierarchy level. If `level = 0` returns all descendants to the end. [UInt8](../../sql-reference/data-types/int-uint.md).

View File

@ -81,7 +81,7 @@ SELECT bitmapToArray(bitmapSubsetInRange(bitmapBuild([0,1,2,3,4,5,6,7,8,9,10,11,
**示例**
``` sql
SELECT bitmapToArray(bitmapSubsetInRange(bitmapBuild([0,1,2,3,4,5,6,7,8,9,10,11,12,13,14,15,16,17,18,19,20,21,22,23,24,25,26,27,28,29,30,31,32,33,100,200,500]), toUInt32(30), toUInt32(200))) AS res
SELECT bitmapToArray(bitmapSubsetLimit(bitmapBuild([0,1,2,3,4,5,6,7,8,9,10,11,12,13,14,15,16,17,18,19,20,21,22,23,24,25,26,27,28,29,30,31,32,33,100,200,500]), toUInt32(30), toUInt32(200))) AS res
```
┌─res───────────────────────┐
@ -174,7 +174,7 @@ SELECT bitmapToArray(bitmapAnd(bitmapBuild([1,2,3]),bitmapBuild([3,4,5]))) AS re
│ [3] │
└─────┘
## 位图 {#bitmapor}
## 位图 {#bitmapor}
为两个位图对象进行或操作,返回一个新的位图对象。

View File

@ -430,6 +430,7 @@ private:
{TokenType::ClosingRoundBracket, Replxx::Color::BROWN},
{TokenType::OpeningSquareBracket, Replxx::Color::BROWN},
{TokenType::ClosingSquareBracket, Replxx::Color::BROWN},
{TokenType::DoubleColon, Replxx::Color::BROWN},
{TokenType::OpeningCurlyBrace, Replxx::Color::INTENSE},
{TokenType::ClosingCurlyBrace, Replxx::Color::INTENSE},

View File

@ -388,24 +388,32 @@ void LocalServer::processQueries()
/// Use the same query_id (and thread group) for all queries
CurrentThread::QueryScope query_scope_holder(context);
///Set progress show
/// Set progress show
need_render_progress = config().getBool("progress", false);
std::function<void()> finalize_progress;
if (need_render_progress)
{
/// Set progress callback, which can be run from multiple threads.
context->setProgressCallback([&](const Progress & value)
{
/// Write progress only if progress was updated
if (progress_indication.updateProgress(value))
progress_indication.writeProgress();
});
/// Set finalizing callback for progress, which is called right before finalizing query output.
finalize_progress = [&]()
{
progress_indication.clearProgressOutput();
};
/// Set callback for file processing progress.
progress_indication.setFileProgressCallback(context);
}
bool echo_queries = config().hasOption("echo") || config().hasOption("verbose");
if (need_render_progress)
progress_indication.setFileProgressCallback(context);
std::exception_ptr exception;
for (const auto & query : queries)
@ -425,7 +433,7 @@ void LocalServer::processQueries()
try
{
executeQuery(read_buf, write_buf, /* allow_into_outfile = */ true, context, {});
executeQuery(read_buf, write_buf, /* allow_into_outfile = */ true, context, {}, finalize_progress);
}
catch (...)
{

View File

@ -298,11 +298,19 @@ void ConfigProcessor::doIncludesRecursive(
{
const auto * subst = attributes->getNamedItem(attr_name);
attr_nodes[attr_name] = subst;
substs_count += static_cast<size_t>(subst == nullptr);
substs_count += static_cast<size_t>(subst != nullptr);
}
if (substs_count < SUBSTITUTION_ATTRS.size() - 1) /// only one substitution is allowed
throw Poco::Exception("several substitutions attributes set for element <" + node->nodeName() + ">");
if (substs_count > 1) /// only one substitution is allowed
throw Poco::Exception("More than one substitution attribute is set for element <" + node->nodeName() + ">");
if (node->nodeName() == "include")
{
if (node->hasChildNodes())
throw Poco::Exception("<include> element must have no children");
if (substs_count == 0)
throw Poco::Exception("No substitution attributes set for element <include>, must have exactly one");
}
/// Replace the original contents, not add to it.
bool replace = attributes->getNamedItem("replace");
@ -320,37 +328,57 @@ void ConfigProcessor::doIncludesRecursive(
else if (throw_on_bad_incl)
throw Poco::Exception(error_msg + name);
else
{
if (node->nodeName() == "include")
node->parentNode()->removeChild(node);
LOG_WARNING(log, "{}{}", error_msg, name);
}
}
else
{
Element & element = dynamic_cast<Element &>(*node);
for (const auto & attr_name : SUBSTITUTION_ATTRS)
element.removeAttribute(attr_name);
if (replace)
/// Replace the whole node not just contents.
if (node->nodeName() == "include")
{
while (Node * child = node->firstChild())
node->removeChild(child);
const NodeListPtr children = node_to_include->childNodes();
for (size_t i = 0, size = children->length(); i < size; ++i)
{
NodePtr new_node = config->importNode(children->item(i), true);
node->parentNode()->insertBefore(new_node, node);
}
element.removeAttribute("replace");
node->parentNode()->removeChild(node);
}
const NodeListPtr children = node_to_include->childNodes();
for (size_t i = 0, size = children->length(); i < size; ++i)
else
{
NodePtr new_node = config->importNode(children->item(i), true);
node->appendChild(new_node);
}
Element & element = dynamic_cast<Element &>(*node);
const NamedNodeMapPtr from_attrs = node_to_include->attributes();
for (size_t i = 0, size = from_attrs->length(); i < size; ++i)
{
element.setAttributeNode(dynamic_cast<Attr *>(config->importNode(from_attrs->item(i), true)));
}
for (const auto & attr_name : SUBSTITUTION_ATTRS)
element.removeAttribute(attr_name);
included_something = true;
if (replace)
{
while (Node * child = node->firstChild())
node->removeChild(child);
element.removeAttribute("replace");
}
const NodeListPtr children = node_to_include->childNodes();
for (size_t i = 0, size = children->length(); i < size; ++i)
{
NodePtr new_node = config->importNode(children->item(i), true);
node->appendChild(new_node);
}
const NamedNodeMapPtr from_attrs = node_to_include->attributes();
for (size_t i = 0, size = from_attrs->length(); i < size; ++i)
{
element.setAttributeNode(dynamic_cast<Attr *>(config->importNode(from_attrs->item(i), true)));
}
included_something = true;
}
}
};

View File

@ -10,16 +10,10 @@ namespace fs = std::filesystem;
namespace DB
{
/// Checks if file exists without throwing an exception but with message in console.
bool safeFsExists(const auto & path)
bool safeFsExists(const String & path)
{
std::error_code ec;
bool res = fs::exists(path, ec);
if (ec)
{
std::cerr << "Can't check '" << path << "': [" << ec.value() << "] " << ec.message() << std::endl;
}
return res;
return fs::exists(path, ec);
};
bool configReadClient(Poco::Util::LayeredConfiguration & config, const std::string & home_path)

View File

@ -4,9 +4,6 @@
#include <Common/UnicodeBar.h>
#include <Databases/DatabaseMemory.h>
/// FIXME: progress bar in clickhouse-local needs to be cleared after query execution
/// - same as it is now in clickhouse-client. Also there is no writeFinalProgress call
/// in clickhouse-local.
namespace DB
{

View File

@ -1,11 +1,13 @@
#include <Columns/ColumnArray.h>
#include <Columns/ColumnConst.h>
#include <Columns/ColumnTuple.h>
#include <Columns/ColumnMap.h>
#include <Columns/ColumnLowCardinality.h>
#include <DataTypes/DataTypeLowCardinality.h>
#include <DataTypes/DataTypeArray.h>
#include <DataTypes/DataTypeTuple.h>
#include <DataTypes/DataTypeMap.h>
#include <Common/assert_cast.h>
@ -39,6 +41,11 @@ DataTypePtr recursiveRemoveLowCardinality(const DataTypePtr & type)
return std::make_shared<DataTypeTuple>(elements);
}
if (const auto * map_type = typeid_cast<const DataTypeMap *>(type.get()))
{
return std::make_shared<DataTypeMap>(recursiveRemoveLowCardinality(map_type->getKeyType()), recursiveRemoveLowCardinality(map_type->getValueType()));
}
if (const auto * low_cardinality_type = typeid_cast<const DataTypeLowCardinality *>(type.get()))
return low_cardinality_type->getDictionaryType();
@ -78,6 +85,16 @@ ColumnPtr recursiveRemoveLowCardinality(const ColumnPtr & column)
return ColumnTuple::create(columns);
}
if (const auto * column_map = typeid_cast<const ColumnMap *>(column.get()))
{
const auto & nested = column_map->getNestedColumnPtr();
auto nested_no_lc = recursiveRemoveLowCardinality(nested);
if (nested.get() == nested_no_lc.get())
return column;
return ColumnMap::create(nested_no_lc);
}
if (const auto * column_low_cardinality = typeid_cast<const ColumnLowCardinality *>(column.get()))
return column_low_cardinality->convertToFullColumn();

View File

@ -7,6 +7,7 @@
#include <DataTypes/DataTypeMap.h>
#include <DataTypes/DataTypeArray.h>
#include <DataTypes/DataTypeTuple.h>
#include <DataTypes/DataTypeLowCardinality.h>
#include <DataTypes/DataTypeFactory.h>
#include <DataTypes/Serializations/SerializationMap.h>
#include <Parsers/IAST.h>
@ -53,12 +54,24 @@ DataTypeMap::DataTypeMap(const DataTypePtr & key_type_, const DataTypePtr & valu
void DataTypeMap::assertKeyType() const
{
if (!key_type->isValueRepresentedByInteger()
bool type_error = false;
if (key_type->getTypeId() == TypeIndex::LowCardinality)
{
const auto & low_cardinality_data_type = assert_cast<const DataTypeLowCardinality &>(*key_type);
if (!isStringOrFixedString(*(low_cardinality_data_type.getDictionaryType())))
type_error = true;
}
else if (!key_type->isValueRepresentedByInteger()
&& !isStringOrFixedString(*key_type)
&& !WhichDataType(key_type).isNothing()
&& !WhichDataType(key_type).isUUID())
{
type_error = true;
}
if (type_error)
throw Exception(ErrorCodes::BAD_ARGUMENTS,
"Type of Map key must be a type, that can be represented by integer or string or UUID,"
"Type of Map key must be a type, that can be represented by integer or String or FixedString (possibly LowCardinality) or UUID,"
" but {} given", key_type->getName());
}

View File

@ -417,7 +417,11 @@ void IDiskRemote::removeDirectory(const String & path)
DiskDirectoryIteratorPtr IDiskRemote::iterateDirectory(const String & path)
{
return std::make_unique<RemoteDiskDirectoryIterator>(metadata_path + path, path);
fs::path meta_path = fs::path(metadata_path) / path;
if (fs::exists(meta_path) && fs::is_directory(meta_path))
return std::make_unique<RemoteDiskDirectoryIterator>(meta_path, path);
else
return std::make_unique<RemoteDiskDirectoryIterator>();
}

View File

@ -193,6 +193,7 @@ struct IDiskRemote::Metadata
class RemoteDiskDirectoryIterator final : public IDiskDirectoryIterator
{
public:
RemoteDiskDirectoryIterator() {}
RemoteDiskDirectoryIterator(const String & full_path, const String & folder_path_) : iter(full_path), folder_path(folder_path_) {}
void next() override { ++iter; }

View File

@ -28,7 +28,7 @@ public:
static constexpr auto name = or_null ? "joinGetOrNull" : "joinGet";
bool useDefaultImplementationForNulls() const override { return false; }
bool useDefaultImplementationForLowCardinalityColumns() const override { return true; }
bool useDefaultImplementationForLowCardinalityColumns() const override { return false; }
bool useDefaultImplementationForConstants() const override { return true; }
ColumnPtr executeImpl(const ColumnsWithTypeAndName & arguments, const DataTypePtr &, size_t input_rows_count) const override;

View File

@ -19,6 +19,7 @@
#include <Functions/FunctionHelpers.h>
#include <Functions/IFunction.h>
#include <Interpreters/Context_fwd.h>
#include <Interpreters/castColumn.h>
#include <IO/WriteHelpers.h>
#include <Common/IPv6ToBinary.h>
#include <Common/formatIPv6.h>
@ -978,7 +979,8 @@ public:
!which.isDateTime64() &&
!which.isUInt() &&
!which.isFloat() &&
!which.isDecimal())
!which.isDecimal() &&
!which.isAggregateFunction())
throw Exception("Illegal type " + arguments[0]->getName() + " of argument of function " + getName(),
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
@ -990,6 +992,15 @@ public:
const IColumn * column = arguments[0].column.get();
ColumnPtr res_column;
WhichDataType which(column->getDataType());
if (which.isAggregateFunction())
{
const ColumnPtr to_string = castColumn(arguments[0], std::make_shared<DataTypeString>());
const auto * str_column = checkAndGetColumn<ColumnString>(to_string.get());
tryExecuteString(str_column, res_column);
return res_column;
}
if (tryExecuteUInt<UInt8>(column, res_column) ||
tryExecuteUInt<UInt16>(column, res_column) ||
tryExecuteUInt<UInt32>(column, res_column) ||

View File

@ -163,13 +163,6 @@ public:
arguments[0]->getName(),
getName());
if (!WhichDataType(arguments[1]).isUInt64() &&
!isTuple(arguments[1]))
throw Exception(ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Illegal type {} of second argument of function {} must be UInt64 or tuple(...)",
arguments[1]->getName(),
getName());
return std::make_shared<DataTypeUInt8>();
}
@ -189,8 +182,8 @@ public:
auto dictionary_key_type = dictionary->getKeyType();
const ColumnWithTypeAndName & key_column_with_type = arguments[1];
const auto key_column = key_column_with_type.column;
const auto key_column_type = WhichDataType(key_column_with_type.type);
auto key_column = key_column_with_type.column;
auto key_column_type = key_column_with_type.type;
ColumnPtr range_col = nullptr;
DataTypePtr range_col_type = nullptr;
@ -214,7 +207,7 @@ public:
if (dictionary_key_type == DictionaryKeyType::simple)
{
if (!key_column_type.isUInt64())
if (!WhichDataType(key_column_type).isUInt64())
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Second argument of function {} must be UInt64 when dictionary is simple. Actual type {}.",
@ -225,24 +218,39 @@ public:
}
else if (dictionary_key_type == DictionaryKeyType::complex)
{
if (!key_column_type.isTuple())
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Second argument of function {} must be tuple when dictionary is complex. Actual type {}.",
getName(),
key_column_with_type.type->getName());
/// Functions in external dictionaries_loader only support full-value (not constant) columns with keys.
ColumnPtr key_column_full = key_column->convertToFullColumnIfConst();
key_column = key_column->convertToFullColumnIfConst();
size_t keys_size = dictionary->getStructure().getKeysSize();
const auto & key_columns = typeid_cast<const ColumnTuple &>(*key_column_full).getColumnsCopy();
const auto & key_types = static_cast<const DataTypeTuple &>(*key_column_with_type.type).getElements();
if (!isTuple(key_column_type))
{
if (keys_size > 1)
{
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Third argument of function {} must be tuple when dictionary is complex and key contains more than 1 attribute."
"Actual type {}.",
getName(),
key_column_type->getName());
}
else
{
Columns tuple_columns = {std::move(key_column)};
key_column = ColumnTuple::create(tuple_columns);
DataTypes tuple_types = {key_column_type};
key_column_type = std::make_shared<DataTypeTuple>(tuple_types);
}
}
const auto & key_columns = assert_cast<const ColumnTuple &>(*key_column).getColumnsCopy();
const auto & key_types = assert_cast<const DataTypeTuple &>(*key_column_type).getElements();
return dictionary->hasKeys(key_columns, key_types);
}
else
{
if (!key_column_type.isUInt64())
if (!WhichDataType(key_column_type).isUInt64())
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Second argument of function {} must be UInt64 when dictionary is range. Actual type {}.",
@ -346,13 +354,6 @@ public:
Strings attribute_names = getAttributeNamesFromColumn(arguments[1].column, arguments[1].type);
auto dictionary = helper.getDictionary(dictionary_name);
if (!WhichDataType(arguments[2].type).isUInt64() && !isTuple(arguments[2].type))
throw Exception(ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Illegal type {} of third argument of function {}, must be UInt64 or tuple(...).",
arguments[2].type->getName(),
getName());
auto dictionary_key_type = dictionary->getKeyType();
size_t current_arguments_index = 3;
@ -446,18 +447,35 @@ public:
}
else if (dictionary_key_type == DictionaryKeyType::complex)
{
if (!isTuple(key_col_with_type.type))
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Third argument of function {} must be tuple when dictionary is complex. Actual type {}.",
getName(),
key_col_with_type.type->getName());
/// Functions in external dictionaries_loader only support full-value (not constant) columns with keys.
ColumnPtr key_column_full = key_col_with_type.column->convertToFullColumnIfConst();
ColumnPtr key_column = key_col_with_type.column->convertToFullColumnIfConst();
DataTypePtr key_column_type = key_col_with_type.type;
const auto & key_columns = typeid_cast<const ColumnTuple &>(*key_column_full).getColumnsCopy();
const auto & key_types = static_cast<const DataTypeTuple &>(*key_col_with_type.type).getElements();
size_t keys_size = dictionary->getStructure().getKeysSize();
if (!isTuple(key_column_type))
{
if (keys_size > 1)
{
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Third argument of function {} must be tuple when dictionary is complex and key contains more than 1 attribute."
"Actual type {}.",
getName(),
key_col_with_type.type->getName());
}
else
{
Columns tuple_columns = {std::move(key_column)};
key_column = ColumnTuple::create(tuple_columns);
DataTypes tuple_types = {key_column_type};
key_column_type = std::make_shared<DataTypeTuple>(tuple_types);
}
}
const auto & key_columns = assert_cast<const ColumnTuple &>(*key_column).getColumnsCopy();
const auto & key_types = assert_cast<const DataTypeTuple &>(*key_column_type).getElements();
result = executeDictionaryRequest(
dictionary,

View File

@ -607,6 +607,8 @@ public:
}
};
template <typename JSONParser>
class JSONExtractRawImpl;
/// Nodes of the extract tree. We need the extract tree to extract from JSON complex values containing array, tuples or nullables.
template <typename JSONParser>
@ -691,7 +693,10 @@ struct JSONExtractTree
public:
bool insertResultToColumn(IColumn & dest, const Element & element) override
{
return JSONExtractStringImpl<JSONParser>::insertResultToColumn(dest, element, {});
if (element.isString())
return JSONExtractStringImpl<JSONParser>::insertResultToColumn(dest, element, {});
else
return JSONExtractRawImpl<JSONParser>::insertResultToColumn(dest, element, {});
}
};

View File

@ -755,6 +755,7 @@ struct GenericValueSource : public ValueSourceImpl<GenericValueSource>
{
using Slice = GenericValueSlice;
using SinkType = GenericArraySink;
using Column = IColumn;
const IColumn * column;
size_t total_rows;

View File

@ -6,6 +6,7 @@
#include <Columns/IColumn.h>
#include <Columns/ColumnVector.h>
#include <Common/typeid_cast.h>
#include <Common/NaNUtils.h>
#include <Common/SipHash.h>
#include <common/range.h>
@ -40,6 +41,7 @@ namespace DB
namespace ErrorCodes
{
extern const int LOGICAL_ERROR;
extern const int BAD_ARGUMENTS;
}
@ -304,6 +306,13 @@ void PointInPolygonWithGrid<CoordinateType>::calcGridAttributes(
y_scale = 1 / cell_height;
x_shift = -min_corner.x();
y_shift = -min_corner.y();
if (!(isFinite(x_scale)
&& isFinite(y_scale)
&& isFinite(x_shift)
&& isFinite(y_shift)
&& isFinite(grid_size)))
throw Exception("Polygon is not valid: bounding box is unbounded", ErrorCodes::BAD_ARGUMENTS);
}
template <typename CoordinateType>
@ -358,7 +367,7 @@ bool PointInPolygonWithGrid<CoordinateType>::contains(CoordinateType x, Coordina
if (has_empty_bound)
return false;
if (std::isnan(x) || std::isnan(y))
if (!isFinite(x) || !isFinite(y))
return false;
CoordinateType float_row = (y + y_shift) * y_scale;

View File

@ -12,6 +12,7 @@
#include <Columns/ColumnNullable.h>
#include <Columns/ColumnsNumber.h>
#include <Columns/ColumnString.h>
#include <Columns/ColumnFixedString.h>
#include <Columns/ColumnTuple.h>
#include <Columns/ColumnMap.h>
#include <Common/typeid_cast.h>
@ -110,6 +111,9 @@ private:
static bool matchKeyToIndexString(const IColumn & data, const Offsets & offsets,
const ColumnsWithTypeAndName & arguments, PaddedPODArray<UInt64> & matched_idxs);
static bool matchKeyToIndexFixedString(const IColumn & data, const Offsets & offsets,
const ColumnsWithTypeAndName & arguments, PaddedPODArray<UInt64> & matched_idxs);
static bool matchKeyToIndexStringConst(const IColumn & data, const Offsets & offsets,
const Field & index, PaddedPODArray<UInt64> & matched_idxs);
@ -767,6 +771,19 @@ struct MatcherString
}
};
struct MatcherFixedString
{
const ColumnFixedString & data;
const ColumnFixedString & index;
bool match(size_t row_data, size_t row_index) const
{
auto data_ref = data.getDataAt(row_data);
auto index_ref = index.getDataAt(row_index);
return memequalSmallAllowOverflow15(index_ref.data, index_ref.size, data_ref.data, data_ref.size);
}
};
struct MatcherStringConst
{
const ColumnString & data;
@ -863,6 +880,23 @@ bool FunctionArrayElement::matchKeyToIndexString(
return true;
}
bool FunctionArrayElement::matchKeyToIndexFixedString(
const IColumn & data, const Offsets & offsets,
const ColumnsWithTypeAndName & arguments, PaddedPODArray<UInt64> & matched_idxs)
{
const auto * index_string = checkAndGetColumn<ColumnFixedString>(arguments[1].column.get());
if (!index_string)
return false;
const auto * data_string = checkAndGetColumn<ColumnFixedString>(&data);
if (!data_string)
return false;
MatcherFixedString matcher{*data_string, *index_string};
executeMatchKeyToIndex(offsets, matched_idxs, matcher);
return true;
}
template <typename DataType>
bool FunctionArrayElement::matchKeyToIndexNumberConst(
const IColumn & data, const Offsets & offsets,
@ -922,8 +956,10 @@ bool FunctionArrayElement::matchKeyToIndex(
|| matchKeyToIndexNumber<Int64>(data, offsets, arguments, matched_idxs)
|| matchKeyToIndexNumber<Int128>(data, offsets, arguments, matched_idxs)
|| matchKeyToIndexNumber<Int256>(data, offsets, arguments, matched_idxs)
|| matchKeyToIndexNumber<UInt256>(data, offsets, arguments, matched_idxs)
|| matchKeyToIndexNumber<UUID>(data, offsets, arguments, matched_idxs)
|| matchKeyToIndexString(data, offsets, arguments, matched_idxs);
|| matchKeyToIndexString(data, offsets, arguments, matched_idxs)
|| matchKeyToIndexFixedString(data, offsets, arguments, matched_idxs);
}
bool FunctionArrayElement::matchKeyToIndexConst(

308
src/Functions/padString.cpp Normal file
View File

@ -0,0 +1,308 @@
#include <Columns/ColumnFixedString.h>
#include <Columns/ColumnString.h>
#include <Functions/FunctionFactory.h>
#include <Functions/FunctionHelpers.h>
#include <Functions/GatherUtils/Algorithms.h>
#include <Functions/GatherUtils/Sinks.h>
#include <Functions/GatherUtils/Sources.h>
#include <common/bit_cast.h>
namespace DB
{
using namespace GatherUtils;
namespace ErrorCodes
{
extern const int ILLEGAL_COLUMN;
extern const int ILLEGAL_TYPE_OF_ARGUMENT;
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
extern const int TOO_LARGE_STRING_SIZE;
}
namespace
{
/// The maximum new padded length.
constexpr size_t MAX_NEW_LENGTH = 1000000;
/// Appends padding characters to a sink based on a pad string.
/// Depending on how many padding characters are required to add
/// the pad string can be copied only partly or be repeated multiple times.
template <bool is_utf8>
class PaddingChars
{
public:
explicit PaddingChars(const String & pad_string_) : pad_string(pad_string_) { init(); }
ALWAYS_INLINE size_t numCharsInPadString() const
{
if constexpr (is_utf8)
return utf8_offsets.size() - 1;
else
return pad_string.length();
}
ALWAYS_INLINE size_t numCharsToNumBytes(size_t count) const
{
if constexpr (is_utf8)
return utf8_offsets[count];
else
return count;
}
void appendTo(StringSink & res_sink, size_t num_chars) const
{
if (!num_chars)
return;
const size_t step = numCharsInPadString();
while (true)
{
if (num_chars <= step)
{
writeSlice(StringSource::Slice{bit_cast<const UInt8 *>(pad_string.data()), numCharsToNumBytes(num_chars)}, res_sink);
break;
}
writeSlice(StringSource::Slice{bit_cast<const UInt8 *>(pad_string.data()), numCharsToNumBytes(step)}, res_sink);
num_chars -= step;
}
}
private:
void init()
{
if (pad_string.empty())
pad_string = " ";
if constexpr (is_utf8)
{
size_t offset = 0;
utf8_offsets.reserve(pad_string.length() + 1);
while (true)
{
utf8_offsets.push_back(offset);
if (offset == pad_string.length())
break;
offset += UTF8::seqLength(pad_string[offset]);
if (offset > pad_string.length())
offset = pad_string.length();
}
}
/// Not necessary, but good for performance.
while (numCharsInPadString() < 16)
{
pad_string += pad_string;
if constexpr (is_utf8)
{
size_t old_size = utf8_offsets.size();
utf8_offsets.reserve((old_size - 1) * 2);
size_t base = utf8_offsets.back();
for (size_t i = 1; i != old_size; ++i)
utf8_offsets.push_back(utf8_offsets[i] + base);
}
}
}
String pad_string;
std::vector<size_t> utf8_offsets;
};
/// Returns the number of characters in a slice.
template <bool is_utf8>
inline ALWAYS_INLINE size_t getLengthOfSlice(const StringSource::Slice & slice)
{
if constexpr (is_utf8)
return UTF8::countCodePoints(slice.data, slice.size);
else
return slice.size;
}
/// Moves the end of a slice back by n characters.
template <bool is_utf8>
inline ALWAYS_INLINE StringSource::Slice removeSuffixFromSlice(const StringSource::Slice & slice, size_t suffix_length)
{
StringSource::Slice res = slice;
if constexpr (is_utf8)
res.size = UTF8StringSource::skipCodePointsBackward(slice.data + slice.size, suffix_length, slice.data) - res.data;
else
res.size -= std::min(suffix_length, res.size);
return res;
}
/// If `is_right_pad` - it's the rightPad() function instead of leftPad().
/// If `is_utf8` - lengths are measured in code points instead of bytes.
template <bool is_right_pad, bool is_utf8>
class FunctionPadString : public IFunction
{
public:
static constexpr auto name = is_right_pad ? (is_utf8 ? "rightPadUTF8" : "rightPad") : (is_utf8 ? "leftPadUTF8" : "leftPad");
static FunctionPtr create(const ContextPtr) { return std::make_shared<FunctionPadString>(); }
String getName() const override { return name; }
bool isVariadic() const override { return true; }
size_t getNumberOfArguments() const override { return 0; }
bool useDefaultImplementationForConstants() const override { return false; }
DataTypePtr getReturnTypeImpl(const DataTypes & arguments) const override
{
size_t number_of_arguments = arguments.size();
if (number_of_arguments != 2 && number_of_arguments != 3)
throw Exception(
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH,
"Number of arguments for function {} doesn't match: passed {}, should be 2 or 3",
getName(),
std::to_string(number_of_arguments));
if (!isStringOrFixedString(arguments[0]))
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Illegal type {} of the first argument of function {}, should be string",
arguments[0]->getName(),
getName());
if (!isUnsignedInteger(arguments[1]))
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Illegal type {} of the second argument of function {}, should be unsigned integer",
arguments[1]->getName(),
getName());
if (number_of_arguments == 3 && !isStringOrFixedString(arguments[2]))
throw Exception(
ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Illegal type {} of the third argument of function {}, should be const string",
arguments[2]->getName(),
getName());
return arguments[0];
}
ColumnPtr executeImpl(const ColumnsWithTypeAndName & arguments, const DataTypePtr &, size_t input_rows_count) const override
{
auto column_string = arguments[0].column;
auto column_length = arguments[1].column;
String pad_string;
if (arguments.size() == 3)
{
auto column_pad = arguments[2].column;
const ColumnConst * column_pad_const = checkAndGetColumnConst<ColumnString>(column_pad.get());
if (!column_pad_const)
throw Exception(
ErrorCodes::ILLEGAL_COLUMN,
"Illegal column {}, third argument of function {} must be a constant string",
column_pad->getName(),
getName());
pad_string = column_pad_const->getValue<String>();
}
PaddingChars<is_utf8> padding_chars{pad_string};
auto col_res = ColumnString::create();
StringSink res_sink{*col_res, input_rows_count};
if (const ColumnString * col = checkAndGetColumn<ColumnString>(column_string.get()))
executeForSource(StringSource{*col}, column_length, padding_chars, res_sink);
else if (const ColumnFixedString * col_fixed = checkAndGetColumn<ColumnFixedString>(column_string.get()))
executeForSource(FixedStringSource{*col_fixed}, column_length, padding_chars, res_sink);
else if (const ColumnConst * col_const = checkAndGetColumnConst<ColumnString>(column_string.get()))
executeForSource(ConstSource<StringSource>{*col_const}, column_length, padding_chars, res_sink);
else if (const ColumnConst * col_const_fixed = checkAndGetColumnConst<ColumnFixedString>(column_string.get()))
executeForSource(ConstSource<FixedStringSource>{*col_const_fixed}, column_length, padding_chars, res_sink);
else
throw Exception(
ErrorCodes::ILLEGAL_COLUMN,
"Illegal column {}, first argument of function {} must be a string",
arguments[0].column->getName(),
getName());
return col_res;
}
private:
template <typename SourceStrings>
void executeForSource(
SourceStrings && strings,
const ColumnPtr & column_length,
const PaddingChars<is_utf8> & padding_chars,
StringSink & res_sink) const
{
if (const auto * col_const = checkAndGetColumn<ColumnConst>(column_length.get()))
executeForSourceAndLength(std::forward<SourceStrings>(strings), ConstSource<GenericValueSource>{*col_const}, padding_chars, res_sink);
else
executeForSourceAndLength(std::forward<SourceStrings>(strings), GenericValueSource{*column_length}, padding_chars, res_sink);
}
template <typename SourceStrings, typename SourceLengths>
void executeForSourceAndLength(
SourceStrings && strings,
SourceLengths && lengths,
const PaddingChars<is_utf8> & padding_chars,
StringSink & res_sink) const
{
bool is_const_length = lengths.isConst();
bool need_check_length = true;
for (; !res_sink.isEnd(); res_sink.next(), strings.next(), lengths.next())
{
auto str = strings.getWhole();
size_t current_length = getLengthOfSlice<is_utf8>(str);
auto new_length_slice = lengths.getWhole();
size_t new_length = new_length_slice.elements->getUInt(new_length_slice.position);
if (need_check_length)
{
if (new_length > MAX_NEW_LENGTH)
{
throw Exception(
"New padded length (" + std::to_string(new_length) + ") is too big, maximum is: " + std::to_string(MAX_NEW_LENGTH),
ErrorCodes::TOO_LARGE_STRING_SIZE);
}
if (is_const_length)
{
size_t rows_count = res_sink.offsets.size();
res_sink.reserve((new_length + 1 /* zero terminator */) * rows_count);
need_check_length = false;
}
}
if (new_length == current_length)
{
writeSlice(str, res_sink);
}
else if (new_length < current_length)
{
str = removeSuffixFromSlice<is_utf8>(str, current_length - new_length);
writeSlice(str, res_sink);
}
else if (new_length > current_length)
{
if constexpr (!is_right_pad)
padding_chars.appendTo(res_sink, new_length - current_length);
writeSlice(str, res_sink);
if constexpr (is_right_pad)
padding_chars.appendTo(res_sink, new_length - current_length);
}
}
}
};
}
void registerFunctionPadString(FunctionFactory & factory)
{
factory.registerFunction<FunctionPadString<false, false>>(); /// leftPad
factory.registerFunction<FunctionPadString<false, true>>(); /// leftPadUTF8
factory.registerFunction<FunctionPadString<true, false>>(); /// rightPad
factory.registerFunction<FunctionPadString<true, true>>(); /// rightPadUTF8
factory.registerAlias("lpad", "leftPad", FunctionFactory::CaseInsensitive);
factory.registerAlias("rpad", "rightPad", FunctionFactory::CaseInsensitive);
}
}

View File

@ -29,6 +29,7 @@ void registerFunctionAppendTrailingCharIfAbsent(FunctionFactory &);
void registerFunctionStartsWith(FunctionFactory &);
void registerFunctionEndsWith(FunctionFactory &);
void registerFunctionTrim(FunctionFactory &);
void registerFunctionPadString(FunctionFactory &);
void registerFunctionRegexpQuoteMeta(FunctionFactory &);
void registerFunctionNormalizeQuery(FunctionFactory &);
void registerFunctionNormalizedQueryHash(FunctionFactory &);
@ -68,6 +69,7 @@ void registerFunctionsString(FunctionFactory & factory)
registerFunctionStartsWith(factory);
registerFunctionEndsWith(factory);
registerFunctionTrim(factory);
registerFunctionPadString(factory);
registerFunctionRegexpQuoteMeta(factory);
registerFunctionNormalizeQuery(factory);
registerFunctionNormalizedQueryHash(factory);

View File

@ -386,6 +386,7 @@ SRCS(
now.cpp
now64.cpp
nullIf.cpp
padString.cpp
partitionId.cpp
pi.cpp
plus.cpp

View File

@ -948,7 +948,8 @@ void executeQuery(
WriteBuffer & ostr,
bool allow_into_outfile,
ContextMutablePtr context,
std::function<void(const String &, const String &, const String &, const String &)> set_result_details)
std::function<void(const String &, const String &, const String &, const String &)> set_result_details,
std::function<void()> before_finalize_callback)
{
PODArray<char> parse_buf;
const char * begin;
@ -1079,6 +1080,8 @@ void executeQuery(
out->onProgress(progress);
});
out->setBeforeFinalizeCallback(before_finalize_callback);
if (set_result_details)
set_result_details(
context->getClientInfo().current_query_id, out->getContentType(), format_name, DateLUT::instance().getTimeZone());

View File

@ -17,7 +17,8 @@ void executeQuery(
WriteBuffer & ostr, /// Where to write query output to.
bool allow_into_outfile, /// If true and the query contains INTO OUTFILE section, redirect output to that file.
ContextMutablePtr context, /// DB, tables, data types, storage engines, functions, aggregate functions...
std::function<void(const String &, const String &, const String &, const String &)> set_result_details /// If a non-empty callback is passed, it will be called with the query id, the content-type, the format, and the timezone.
std::function<void(const String &, const String &, const String &, const String &)> set_result_details, /// If a non-empty callback is passed, it will be called with the query id, the content-type, the format, and the timezone.
std::function<void()> before_finalize_callback = {} /// Will be set in output format to be called before finalize.
);

View File

@ -1,4 +1,5 @@
#include <memory>
#include <Parsers/ASTLiteral.h>
#include <Parsers/ASTSelectQuery.h>
#include <Parsers/IParserBase.h>
#include <Parsers/CommonParsers.h>
@ -16,11 +17,12 @@ namespace DB
namespace ErrorCodes
{
extern const int TOP_AND_LIMIT_TOGETHER;
extern const int WITH_TIES_WITHOUT_ORDER_BY;
extern const int FIRST_AND_NEXT_TOGETHER;
extern const int LIMIT_BY_WITH_TIES_IS_NOT_SUPPORTED;
extern const int ROW_AND_ROWS_TOGETHER;
extern const int FIRST_AND_NEXT_TOGETHER;
extern const int SYNTAX_ERROR;
extern const int TOP_AND_LIMIT_TOGETHER;
extern const int WITH_TIES_WITHOUT_ORDER_BY;
}
@ -32,6 +34,7 @@ bool ParserSelectQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
ParserKeyword s_select("SELECT");
ParserKeyword s_all("ALL");
ParserKeyword s_distinct("DISTINCT");
ParserKeyword s_distinct_on("DISTINCT ON");
ParserKeyword s_from("FROM");
ParserKeyword s_prewhere("PREWHERE");
ParserKeyword s_where("WHERE");
@ -77,12 +80,13 @@ bool ParserSelectQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
ASTPtr limit_by_length;
ASTPtr limit_by_offset;
ASTPtr limit_by_expression_list;
ASTPtr distinct_on_expression_list;
ASTPtr limit_offset;
ASTPtr limit_length;
ASTPtr top_length;
ASTPtr settings;
/// WITH expr list
/// WITH expr_list
{
if (s_with.ignore(pos, expected))
{
@ -94,7 +98,7 @@ bool ParserSelectQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
}
}
/// SELECT [ALL/DISTINCT] [TOP N [WITH TIES]] expr list
/// SELECT [ALL/DISTINCT [ON (expr_list)]] [TOP N [WITH TIES]] expr_list
{
bool has_all = false;
if (!s_select.ignore(pos, expected))
@ -103,13 +107,27 @@ bool ParserSelectQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
if (s_all.ignore(pos, expected))
has_all = true;
if (s_distinct.ignore(pos, expected))
if (s_distinct_on.ignore(pos, expected))
{
if (open_bracket.ignore(pos, expected))
{
if (!exp_list.parse(pos, distinct_on_expression_list, expected))
return false;
if (!close_bracket.ignore(pos, expected))
return false;
}
else
return false;
}
else if (s_distinct.ignore(pos, expected))
{
select_query->distinct = true;
}
if (!has_all && s_all.ignore(pos, expected))
has_all = true;
if (has_all && select_query->distinct)
if (has_all && (select_query->distinct || distinct_on_expression_list))
return false;
if (s_top.ignore(pos, expected))
@ -256,13 +274,19 @@ bool ParserSelectQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
select_query->limit_with_ties = true;
}
if (limit_with_ties_occured && distinct_on_expression_list)
throw Exception("Can not use WITH TIES alongside LIMIT BY/DISTINCT ON", ErrorCodes::LIMIT_BY_WITH_TIES_IS_NOT_SUPPORTED);
if (s_by.ignore(pos, expected))
{
/// WITH TIES was used alongside LIMIT BY
/// But there are other kind of queries like LIMIT n BY smth LIMIT m WITH TIES which are allowed.
/// So we have to ignore WITH TIES exactly in LIMIT BY state.
if (limit_with_ties_occured)
throw Exception("Can not use WITH TIES alongside LIMIT BY", ErrorCodes::LIMIT_BY_WITH_TIES_IS_NOT_SUPPORTED);
throw Exception("Can not use WITH TIES alongside LIMIT BY/DISTINCT ON", ErrorCodes::LIMIT_BY_WITH_TIES_IS_NOT_SUPPORTED);
if (distinct_on_expression_list)
throw Exception("Can not use DISTINCT ON alongside LIMIT BY", ErrorCodes::SYNTAX_ERROR);
limit_by_length = limit_length;
limit_by_offset = limit_offset;
@ -335,6 +359,17 @@ bool ParserSelectQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
}
}
if (distinct_on_expression_list)
{
/// DISTINCT ON and LIMIT BY are mutually exclusive, checked before
assert (limit_by_expression_list == nullptr);
/// Transform `DISTINCT ON expr` to `LIMIT 1 BY expr`
limit_by_expression_list = distinct_on_expression_list;
limit_by_length = std::make_shared<ASTLiteral>(Field{UInt8(1)});
distinct_on_expression_list = nullptr;
}
/// Because TOP n in totally equals LIMIT n
if (top_length)
limit_length = top_length;

View File

@ -76,6 +76,9 @@ void IOutputFormat::work()
if (rows_before_limit_counter && rows_before_limit_counter->hasAppliedLimit())
setRowsBeforeLimit(rows_before_limit_counter->get());
if (before_finalize_callback)
before_finalize_callback();
finalize();
finalized = true;
return;
@ -117,4 +120,3 @@ void IOutputFormat::write(const Block & block)
}
}

View File

@ -67,6 +67,9 @@ public:
/// Passed value are delta, that must be summarized.
virtual void onProgress(const Progress & /*progress*/) {}
/// Set callback, which will be called before call to finalize().
void setBeforeFinalizeCallback(std::function<void()> callback) { before_finalize_callback = callback; }
/// Content-Type to set when sending HTTP response.
virtual std::string getContentType() const { return "text/plain; charset=UTF-8"; }
@ -91,6 +94,7 @@ private:
size_t result_bytes = 0;
bool prefix_written = false;
std::function<void()> before_finalize_callback;
};
}

View File

@ -103,7 +103,7 @@ void printPipelineCompact(const Processors & processors, WriteBuffer & out, bool
out << "digraph\n{\n";
out << " rankdir=\"LR\";\n";
out << " { node [shape = box]\n";
out << " { node [shape = rect]\n";
/// Nodes // TODO quoting and escaping
size_t next_step = 0;

View File

@ -16,7 +16,7 @@ void printPipeline(const Processors & processors, const Statuses & statuses, Wri
{
out << "digraph\n{\n";
out << " rankdir=\"LR\";\n";
out << " { node [shape = box]\n";
out << " { node [shape = rect]\n";
auto get_proc_id = [](const IProcessor & proc) -> UInt64
{

View File

@ -4,7 +4,7 @@
#include <IO/WriteHelpers.h>
#include <Common/StatusInfo.h>
#include <boost/algorithm/string/replace.hpp>
#include <regex>
namespace
{
@ -24,9 +24,13 @@ void writeOutLine(DB::WriteBuffer & wb, T && val, TArgs &&... args)
writeOutLine(wb, std::forward<TArgs>(args)...);
}
void replaceInvalidChars(std::string & metric_name)
/// Returns false if name is not valid
bool replaceInvalidChars(std::string & metric_name)
{
std::replace(metric_name.begin(), metric_name.end(), '.', '_');
/// dirty solution
metric_name = std::regex_replace(metric_name, std::regex("[^a-zA-Z0-9_:]"), "_");
metric_name = std::regex_replace(metric_name, std::regex("^[^a-zA-Z]*"), "");
return !metric_name.empty();
}
}
@ -57,7 +61,8 @@ void PrometheusMetricsWriter::write(WriteBuffer & wb) const
std::string metric_name{ProfileEvents::getName(static_cast<ProfileEvents::Event>(i))};
std::string metric_doc{ProfileEvents::getDocumentation(static_cast<ProfileEvents::Event>(i))};
replaceInvalidChars(metric_name);
if (!replaceInvalidChars(metric_name))
continue;
std::string key{profile_events_prefix + metric_name};
writeOutLine(wb, "# HELP", key, metric_doc);
@ -75,7 +80,8 @@ void PrometheusMetricsWriter::write(WriteBuffer & wb) const
std::string metric_name{CurrentMetrics::getName(static_cast<CurrentMetrics::Metric>(i))};
std::string metric_doc{CurrentMetrics::getDocumentation(static_cast<CurrentMetrics::Metric>(i))};
replaceInvalidChars(metric_name);
if (!replaceInvalidChars(metric_name))
continue;
std::string key{current_metrics_prefix + metric_name};
writeOutLine(wb, "# HELP", key, metric_doc);
@ -91,7 +97,8 @@ void PrometheusMetricsWriter::write(WriteBuffer & wb) const
{
std::string key{asynchronous_metrics_prefix + name_value.first};
replaceInvalidChars(key);
if (!replaceInvalidChars(key))
continue;
auto value = name_value.second;
// TODO: add HELP section? asynchronous_metrics contains only key and value
@ -108,7 +115,8 @@ void PrometheusMetricsWriter::write(WriteBuffer & wb) const
std::string metric_name{CurrentStatusInfo::getName(static_cast<CurrentStatusInfo::Status>(i))};
std::string metric_doc{CurrentStatusInfo::getDocumentation(static_cast<CurrentStatusInfo::Status>(i))};
replaceInvalidChars(metric_name);
if (!replaceInvalidChars(metric_name))
continue;
std::string key{current_status_prefix + metric_name};
writeOutLine(wb, "# HELP", key, metric_doc);

View File

@ -95,6 +95,7 @@ const char * auto_contributors[] {
"Anatoly Pugachev",
"ana-uvarova",
"AnaUvarova",
"Andreas Hunkeler",
"AndreevDm",
"Andrei Bodrov",
"Andrei Chulkov",
@ -280,6 +281,7 @@ const char * auto_contributors[] {
"Dongdong Yang",
"DoomzD",
"Dr. Strange Looker",
"d.v.semenov",
"eaxdev",
"eejoin",
"egatov",
@ -290,6 +292,7 @@ const char * auto_contributors[] {
"Eldar Zaitov",
"Elena Baskakova",
"elenaspb2019",
"elevankoff",
"Elghazal Ahmed",
"Elizaveta Mironyuk",
"emakarov",
@ -434,6 +437,7 @@ const char * auto_contributors[] {
"Ivan Starkov",
"ivanzhukov",
"Ivan Zhukov",
"Jack Song",
"JackyWoo",
"Jacob Hayes",
"jakalletti",
@ -476,6 +480,7 @@ const char * auto_contributors[] {
"Konstantin Lebedev",
"Konstantin Malanchev",
"Konstantin Podshumok",
"Konstantin Rudenskii",
"Korenevskiy Denis",
"Korviakov Andrey",
"koshachy",
@ -488,6 +493,7 @@ const char * auto_contributors[] {
"kshvakov",
"kssenii",
"l",
"l1tsolaiki",
"lalex",
"Latysheva Alexandra",
"lehasm",
@ -515,6 +521,7 @@ const char * auto_contributors[] {
"long2ice",
"Lopatin Konstantin",
"Loud_Scream",
"ltybc-coder",
"luc1ph3r",
"Lucid Dreams",
"Luis Bosque",
@ -633,6 +640,7 @@ const char * auto_contributors[] {
"nicelulu",
"Nickita",
"Nickolay Yastrebov",
"nickzhwang",
"Nicolae Vartolomei",
"Nico Mandery",
"Nico Piderman",
@ -871,6 +879,7 @@ const char * auto_contributors[] {
"Veselkov Konstantin",
"vic",
"vicdashkov",
"Victor",
"Victor Tarnavsky",
"Viktor Taranenko",
"vinity",
@ -947,6 +956,7 @@ const char * auto_contributors[] {
"Yuriy Korzhenevskiy",
"Yury Karpovich",
"Yury Stankevich",
"ywill3",
"zamulla",
"zhang2014",
"zhangshengyu",
@ -957,11 +967,13 @@ const char * auto_contributors[] {
"Zhichun Wu",
"Zhipeng",
"zhukai",
"Zijie Lu",
"zlx19950903",
"Zoran Pandovski",
"zvonand",
"zvrr",
"zvvr",
"zxc111",
"zzsmdfj",
"Артем Стрельцов",
"Владислав Тихонов",
@ -980,6 +992,7 @@ const char * auto_contributors[] {
"张风啸",
"徐炘",
"曲正鹏",
"未来星___费",
"极客青年",
"谢磊",
"贾顺名(Jarvis)",

View File

@ -473,17 +473,17 @@ def execute_task(started_cluster, task, cmd_options):
# Tests
@pytest.mark.timeout(600)
@pytest.mark.skip(reason="Too flaky :(")
def test_different_schema(started_cluster):
execute_task(started_cluster, TaskWithDifferentSchema(started_cluster), [])
@pytest.mark.timeout(600)
@pytest.mark.skip(reason="Too flaky :(")
def test_ttl_columns(started_cluster):
execute_task(started_cluster, TaskTTL(started_cluster), [])
@pytest.mark.timeout(600)
@pytest.mark.skip(reason="Too flaky :(")
def test_skip_index(started_cluster):
execute_task(started_cluster, TaskSkipIndex(started_cluster), [])

View File

@ -10,5 +10,8 @@
<profile>default</profile>
<quota>default</quota>
</default>
<include incl="users_1" />
<include incl="users_2" />
</users>
</yandex>

View File

@ -1,5 +1,5 @@
<yandex>
<include_from>/etc/clickhouse-server/config.d/max_query_size.xml</include_from>
<include_from>/etc/clickhouse-server/config.d/include_from_source.xml</include_from>
<profiles>
<default>
<max_query_size incl="mqs" />
@ -11,5 +11,8 @@
<profile>default</profile>
<quota>default</quota>
</default>
<include incl="users_1" />
<include incl="users_2" />
</users>
</yandex>

View File

@ -11,5 +11,7 @@
<profile>default</profile>
<quota>default</quota>
</default>
<include incl="node_does_not_exist" />
</users>
</yandex>

View File

@ -10,5 +10,8 @@
<profile>default</profile>
<quota>default</quota>
</default>
<include from_zk="/users_from_zk_1" />
<include from_zk="/users_from_zk_2" />
</users>
</yandex>

View File

@ -0,0 +1,17 @@
<yandex>
<mqs>99999</mqs>
<users_1>
<user_1>
<password></password>
<profile>default</profile>
</user_1>
</users_1>
<users_2>
<user_2>
<password></password>
<profile>default</profile>
</user_2>
</users_2>
</yandex>

View File

@ -1,3 +0,0 @@
<yandex>
<mqs>99999</mqs>
</yandex>

View File

@ -8,11 +8,11 @@ node2 = cluster.add_instance('node2', user_configs=['configs/config_env.xml'],
env_variables={"MAX_QUERY_SIZE": "55555"})
node3 = cluster.add_instance('node3', user_configs=['configs/config_zk.xml'], with_zookeeper=True)
node4 = cluster.add_instance('node4', user_configs=['configs/config_incl.xml'],
main_configs=['configs/max_query_size.xml']) # include value 77777
main_configs=['configs/include_from_source.xml']) # include value 77777
node5 = cluster.add_instance('node5', user_configs=['configs/config_allow_databases.xml'])
node6 = cluster.add_instance('node6', user_configs=['configs/config_include_from_env.xml'],
env_variables={"INCLUDE_FROM_ENV": "/etc/clickhouse-server/config.d/max_query_size.xml"},
main_configs=['configs/max_query_size.xml'])
env_variables={"INCLUDE_FROM_ENV": "/etc/clickhouse-server/config.d/include_from_source.xml"},
main_configs=['configs/include_from_source.xml'])
@pytest.fixture(scope="module")
@ -20,6 +20,8 @@ def start_cluster():
try:
def create_zk_roots(zk):
zk.create(path="/setting/max_query_size", value=b"77777", makepath=True)
zk.create(path="/users_from_zk_1", value=b"<user_1><password></password><profile>default</profile></user_1>", makepath=True)
zk.create(path="/users_from_zk_2", value=b"<user_2><password></password><profile>default</profile></user_2>", makepath=True)
cluster.add_zookeeper_startup_command(create_zk_roots)
@ -37,6 +39,18 @@ def test_config(start_cluster):
assert node6.query("select value from system.settings where name = 'max_query_size'") == "99999\n"
def test_include_config(start_cluster):
# <include incl="source tag" />
assert node4.query("select 1")
assert node4.query("select 1", user="user_1")
assert node4.query("select 1", user="user_2")
# <include from_zk="zk path />
assert node3.query("select 1")
assert node3.query("select 1", user="user_1")
assert node3.query("select 1", user="user_2")
def test_allow_databases(start_cluster):
node5.query("CREATE DATABASE db1")
node5.query(

View File

@ -67,7 +67,7 @@ hello
1234567890.12345677879616925706 (1234567890.12345677879616925706,'test')
1234567890.123456695758468374595199311875 (1234567890.123456695758468374595199311875,'test')
--JSONExtractKeysAndValues--
[('a','hello')]
[('a','hello'),('b','[-100,200,300]')]
[('b',[-100,200,300])]
[('a','hello'),('b','world')]
[('a',5),('b',7),('c',11)]
@ -170,7 +170,7 @@ Friday
(3,5)
(3,0)
--JSONExtractKeysAndValues--
[('a','hello')]
[('a','hello'),('b','[-100,200,300]')]
[('b',[-100,200,300])]
[('a','hello'),('b','world')]
[('a',5),('b',7),('c',11)]

View File

@ -1 +1,6 @@
yyy
1
1
1
1
1
1

View File

@ -1,9 +1,14 @@
drop table if exists join_tbl;
DROP TABLE IF EXISTS join_tbl;
create table join_tbl (`id` String, `name` String) engine Join(any, left, id);
CREATE TABLE join_tbl (`id` String, `name` String, lcname LowCardinality(String)) ENGINE = Join(any, left, id);
insert into join_tbl values ('xxx', 'yyy');
INSERT INTO join_tbl VALUES ('xxx', 'yyy', 'yyy');
select joinGet('join_tbl', 'name', toLowCardinality('xxx'));
SELECT joinGet('join_tbl', 'name', 'xxx') == 'yyy';
SELECT joinGet('join_tbl', 'name', toLowCardinality('xxx')) == 'yyy';
SELECT joinGet('join_tbl', 'name', toLowCardinality(materialize('xxx'))) == 'yyy';
SELECT joinGet('join_tbl', 'lcname', 'xxx') == 'yyy';
SELECT joinGet('join_tbl', 'lcname', toLowCardinality('xxx')) == 'yyy';
SELECT joinGet('join_tbl', 'lcname', toLowCardinality(materialize('xxx'))) == 'yyy';
drop table if exists join_tbl;
DROP TABLE IF EXISTS join_tbl;

View File

@ -0,0 +1,2 @@
b
{'1':1} 1 0

View File

@ -0,0 +1,12 @@
DROP TABLE IF EXISTS map_lc;
SET allow_experimental_map_type = 1;
CREATE TABLE map_lc
(
`kv` Map(LowCardinality(String), LowCardinality(String))
)
ENGINE = Memory;
INSERT INTO map_lc select map('a', 'b');
SELECT kv['a'] FROM map_lc;
DROP TABLE map_lc;
SELECT map(toFixedString('1',1),1) AS m, m[toFixedString('1',1)],m[toFixedString('1',2)];

View File

@ -0,0 +1 @@
('123','456','[7,8,9]')

View File

@ -0,0 +1 @@
select JSONExtract('{"a": "123", "b": 456, "c": [7, 8, 9]}', 'Tuple(a String, b String, c String)');

View File

@ -0,0 +1,8 @@
1 1 1
2 2 2
1 2 2
1 1 1
2 2 2
1 2 2
1 1 1
2 2 2

View File

@ -0,0 +1,23 @@
DROP TABLE IF EXISTS t1;
CREATE TABLE t1 (`a` UInt32, `b` UInt32, `c` UInt32 ) ENGINE = Memory;
INSERT INTO t1 VALUES (1, 1, 1), (1, 1, 2), (2, 2, 2), (1, 2, 2);
SELECT DISTINCT ON (a, b) a, b, c FROM t1;
SELECT DISTINCT ON (a, b) * FROM t1;
SELECT DISTINCT ON (a) * FROM t1;
-- fuzzer will fail, enable when fixed
-- SELECT DISTINCT ON (a, b) a, b, c FROM t1 LIMIT 1 BY a, b; -- { clientError 62 }
-- SELECT DISTINCT ON a, b a, b FROM t1; -- { clientError 62 }
-- SELECT DISTINCT ON a a, b FROM t1; -- { clientError 62 }
-- "Code: 47. DB::Exception: Missing columns: 'DISTINCT'" - error can be better
-- SELECT DISTINCT ON (a, b) DISTINCT a, b FROM t1; -- { serverError 47 }
-- SELECT DISTINCT DISTINCT ON (a, b) a, b FROM t1; -- { clientError 62 }
-- SELECT ALL DISTINCT ON (a, b) a, b FROM t1; -- { clientError 62 }
-- SELECT DISTINCT ON (a, b) ALL a, b FROM t1; -- { clientError 62 }
DROP TABLE IF EXISTS t1;

View File

@ -33,3 +33,7 @@
1
1
1
1
1
2D000000000000000A
001011010000000000000000000000000000000000000000000000000000000000001010

View File

@ -37,3 +37,9 @@ select bin(unbin('0')) == '00000000';
select hex('') == bin('');
select unhex('') == unbin('');
select unhex('0') == unbin('0');
-- hex and bin support AggregateFunction
select hex(sumState(number)) == hex(toString(sumState(number))) from numbers(10);
select hex(avgState(number)) == hex(toString(avgState(number))) from numbers(99);
select hex(avgState(number)) from numbers(10);
select bin(avgState(number)) from numbers(10);

View File

@ -0,0 +1,54 @@
leftPad
a
ab
abc
abc
abc
abc
ab
*abc
**abc
*******abc
ab
*abc
*.abc
*.*.*.*abc
leftPadUTF8
а
аб
аб
абвг
ЧАабвг
ЧАСЧАСЧАабвг
rightPad
a
ab
abc
abc
abc
abc
ab
abc*
abc**
abc*******
ab
abc*
abc*.
abc*.*.*.*
rightPadUTF8
а
аб
аб
абвг
абвгЧА
абвгЧАСЧАСЧА
numbers
1^
_2^^
__3^^^
___4^^^^
____5^^^^^
_____6^^^^^^

View File

@ -0,0 +1,54 @@
SELECT 'leftPad';
SELECT leftPad('abc', 0);
SELECT leftPad('abc', 1);
SELECT leftPad('abc', 2);
SELECT leftPad('abc', 3);
SELECT leftPad('abc', 4);
SELECT leftPad('abc', 5);
SELECT leftPad('abc', 10);
SELECT leftPad('abc', 2, '*');
SELECT leftPad('abc', 4, '*');
SELECT leftPad('abc', 5, '*');
SELECT leftPad('abc', 10, '*');
SELECT leftPad('abc', 2, '*.');
SELECT leftPad('abc', 4, '*.');
SELECT leftPad('abc', 5, '*.');
SELECT leftPad('abc', 10, '*.');
SELECT 'leftPadUTF8';
SELECT leftPad('абвг', 2);
SELECT leftPadUTF8('абвг', 2);
SELECT leftPad('абвг', 4);
SELECT leftPadUTF8('абвг', 4);
SELECT leftPad('абвг', 12, 'ЧАС');
SELECT leftPadUTF8('абвг', 12, 'ЧАС');
SELECT 'rightPad';
SELECT rightPad('abc', 0);
SELECT rightPad('abc', 1);
SELECT rightPad('abc', 2);
SELECT rightPad('abc', 3);
SELECT rightPad('abc', 4);
SELECT rightPad('abc', 5);
SELECT rightPad('abc', 10);
SELECT rightPad('abc', 2, '*');
SELECT rightPad('abc', 4, '*');
SELECT rightPad('abc', 5, '*');
SELECT rightPad('abc', 10, '*');
SELECT rightPad('abc', 2, '*.');
SELECT rightPad('abc', 4, '*.');
SELECT rightPad('abc', 5, '*.');
SELECT rightPad('abc', 10, '*.');
SELECT 'rightPadUTF8';
SELECT rightPad('абвг', 2);
SELECT rightPadUTF8('абвг', 2);
SELECT rightPad('абвг', 4);
SELECT rightPadUTF8('абвг', 4);
SELECT rightPad('абвг', 12, 'ЧАС');
SELECT rightPadUTF8('абвг', 12, 'ЧАС');
SELECT 'numbers';
SELECT rightPad(leftPad(toString(number), number, '_'), number*2, '^') FROM numbers(7);

View File

@ -0,0 +1,2 @@
SET validate_polygons = 0;
SELECT pointInPolygon((-inf, 1023), [(10.000100135803223, 10000000000.), (inf, 0.9998999834060669), (1.1920928955078125e-7, 100.0000991821289), (1.000100016593933, 100.0000991821289)]);

View File

@ -0,0 +1,10 @@
dictGet
Value
Value
Value
Value
dictHas
1
1
1
1

View File

@ -0,0 +1,26 @@
DROP TABLE IF EXISTS test_dictionary_source;
CREATE TABLE test_dictionary_source (key String, value String) ENGINE=TinyLog;
INSERT INTO test_dictionary_source VALUES ('Key', 'Value');
DROP DICTIONARY IF EXISTS test_dictionary;
CREATE DICTIONARY test_dictionary(key String, value String)
PRIMARY KEY key
LAYOUT(COMPLEX_KEY_HASHED())
SOURCE(CLICKHOUSE(TABLE 'test_dictionary_source'))
LIFETIME(0);
SELECT 'dictGet';
SELECT dictGet('test_dictionary', 'value', tuple('Key'));
SELECT dictGet('test_dictionary', 'value', tuple(materialize('Key')));
SELECT dictGet('test_dictionary', 'value', 'Key');
SELECT dictGet('test_dictionary', 'value', materialize('Key'));
SELECT 'dictHas';
SELECT dictHas('test_dictionary', tuple('Key'));
SELECT dictHas('test_dictionary', tuple(materialize('Key')));
SELECT dictHas('test_dictionary', 'Key');
SELECT dictHas('test_dictionary', materialize('Key'));
DROP DICTIONARY test_dictionary;
DROP TABLE test_dictionary_source;

View File

@ -1,3 +1,4 @@
v21.7.2.7-stable 2021-07-09
v21.6.6.51-stable 2021-07-02
v21.6.5.37-stable 2021-06-19
v21.6.4.26-stable 2021-06-11

1 v21.6.6.51-stable v21.7.2.7-stable 2021-07-02 2021-07-09
1 v21.7.2.7-stable 2021-07-09
2 v21.6.6.51-stable v21.6.6.51-stable 2021-07-02 2021-07-02
3 v21.6.5.37-stable v21.6.5.37-stable 2021-06-19 2021-06-19
4 v21.6.4.26-stable v21.6.4.26-stable 2021-06-11 2021-06-11