Merge remote-tracking branch 'origin/master' into yandex-to-clickhouse-in-configs

This commit is contained in:
Alexey Milovidov 2021-09-22 02:08:29 +03:00
commit 4ade6cc5da
21 changed files with 443 additions and 24 deletions

7
debian/control vendored
View File

@ -5,12 +5,9 @@ Maintainer: Alexey Milovidov <milovidov@clickhouse.com>
Build-Depends: debhelper (>= 9),
cmake | cmake3,
ninja-build,
clang-11,
llvm-11,
clang-13,
llvm-13,
libc6-dev,
libicu-dev,
libreadline-dev,
gperf,
tzdata
Standards-Version: 3.9.8

View File

@ -6,4 +6,4 @@ toc_title: Cloud
# ClickHouse Cloud Service {#clickhouse-cloud-service}
!!! info "Info"
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](/company/#contact) to learn more.
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](https://clickhouse.com/company/#contact) to learn more.

View File

@ -6,4 +6,4 @@ toc_title: Support
# ClickHouse Commercial Support Service {#clickhouse-commercial-support-service}
!!! info "Info"
Detailed public description for ClickHouse support services is not ready yet, please [contact us](/company/#contact) to learn more.
Detailed public description for ClickHouse support services is not ready yet, please [contact us](https://clickhouse.com/company/#contact) to learn more.

View File

@ -114,15 +114,25 @@ To do so, create the `/Library/LaunchDaemons/limit.maxfiles.plist` file with the
</plist>
```
Execute the following command:
Give the file correct permissions:
``` bash
sudo chown root:wheel /Library/LaunchDaemons/limit.maxfiles.plist
```
Reboot.
Validate that the file is correct:
To check if its working, you can use `ulimit -n` command.
``` bash
plutil /Library/LaunchDaemons/limit.maxfiles.plist
```
Load the file (or reboot):
``` bash
sudo launchctl load -w /Library/LaunchDaemons/limit.maxfiles.plist
```
To check if its working, use the `ulimit -n` or `launchctl limit maxfiles` commands.
## Run ClickHouse server:

View File

@ -8,4 +8,4 @@ toc_title: "\u30AF\u30E9\u30A6\u30C9"
# ClickHouse Cloud Service {#clickhouse-cloud-service}
!!! info "Info"
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](/company/#contact) to learn more.
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](https://clickhouse.com/company/#contact) to learn more.

View File

@ -6,4 +6,4 @@ toc_title: "Поставщики облачных услуг ClickHouse"
# Поставщики облачных услуг ClickHouse {#clickhouse-cloud-service-providers}
!!! info "Info"
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](/company/#contact) to learn more.
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](https://clickhouse.com/company/#contact) to learn more.

View File

@ -8,4 +8,4 @@ toc_title: 云
# ClickHouse Cloud Service {#clickhouse-cloud-service}
!!! info "Info"
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](/company/#contact) to learn more.
Detailed public description for ClickHouse cloud services is not ready yet, please [contact us](https://clickhouse.com/company/#contact) to learn more.

View File

@ -6,4 +6,4 @@ toc_title: 支持
# ClickHouse 商业支持服务提供商 {#clickhouse-commercial-support-service-providers}
!!! info "Info"
Detailed public description for ClickHouse support services is not ready yet, please [contact us](/company/#contact) to learn more.
Detailed public description for ClickHouse support services is not ready yet, please [contact us](https://clickhouse.com/company/#contact) to learn more.

View File

@ -0,0 +1,98 @@
#include <AggregateFunctions/IAggregateFunction.h>
#include <AggregateFunctions/AggregateFunctionFactory.h>
#include <AggregateFunctions/FactoryHelpers.h>
#include <Common/ExponentiallySmoothedCounter.h>
#include <Common/FieldVisitorConvertToNumber.h>
#include <DataTypes/DataTypesNumber.h>
#include <IO/ReadHelpers.h>
#include <IO/WriteHelpers.h>
namespace DB
{
namespace ErrorCodes
{
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
extern const int ILLEGAL_TYPE_OF_ARGUMENT;
}
/** See the comments in ExponentiallySmoothedCounter.h
*/
class AggregateFunctionExponentialMovingAverage final
: public IAggregateFunctionDataHelper<ExponentiallySmoothedAverage, AggregateFunctionExponentialMovingAverage>
{
private:
String name;
Float64 half_decay;
public:
AggregateFunctionExponentialMovingAverage(const DataTypes & argument_types_, const Array & params)
: IAggregateFunctionDataHelper<ExponentiallySmoothedAverage, AggregateFunctionExponentialMovingAverage>(argument_types_, params)
{
if (params.size() != 1)
throw Exception{"Aggregate function " + getName() + " requires exactly one parameter: half decay time.",
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH};
half_decay = applyVisitor(FieldVisitorConvertToNumber<Float64>(), params[0]);
}
String getName() const override
{
return "exponentialMovingAverage";
}
DataTypePtr getReturnType() const override
{
return std::make_shared<DataTypeNumber<Float64>>();
}
bool allocatesMemoryInArena() const override { return false; }
void add(AggregateDataPtr __restrict place, const IColumn ** columns, size_t row_num, Arena *) const override
{
const auto & value = columns[0]->getFloat64(row_num);
const auto & time = columns[1]->getFloat64(row_num);
this->data(place).add(value, time, half_decay);
}
void merge(AggregateDataPtr __restrict place, ConstAggregateDataPtr rhs, Arena *) const override
{
this->data(place).merge(this->data(rhs), half_decay);
}
void serialize(ConstAggregateDataPtr __restrict place, WriteBuffer & buf) const override
{
writeBinary(this->data(place).value, buf);
writeBinary(this->data(place).time, buf);
}
void deserialize(AggregateDataPtr __restrict place, ReadBuffer & buf, Arena *) const override
{
readBinary(this->data(place).value, buf);
readBinary(this->data(place).time, buf);
}
void insertResultInto(AggregateDataPtr __restrict place, IColumn & to, Arena *) const override
{
auto & column = assert_cast<ColumnVector<Float64> &>(to);
column.getData().push_back(this->data(place).get(half_decay));
}
};
void registerAggregateFunctionExponentialMovingAverage(AggregateFunctionFactory & factory)
{
factory.registerFunction("exponentialMovingAverage",
[](const std::string & name, const DataTypes & argument_types, const Array & params, const Settings *) -> AggregateFunctionPtr
{
assertBinary(name, argument_types);
for (const auto & type : argument_types)
if (!isNumber(*type))
throw Exception(ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
"Both arguments for aggregate function {} must have numeric type, got {}", name, type->getName());
return std::make_shared<AggregateFunctionExponentialMovingAverage>(argument_types, params);
});
}
}

View File

@ -50,7 +50,9 @@ void registerAggregateFunctionWelchTTest(AggregateFunctionFactory &);
void registerAggregateFunctionStudentTTest(AggregateFunctionFactory &);
void registerAggregateFunctionSingleValueOrNull(AggregateFunctionFactory &);
void registerAggregateFunctionSequenceNextNode(AggregateFunctionFactory &);
void registerAggregateFunctionExponentialMovingAverage(AggregateFunctionFactory &);
void registerAggregateFunctionSparkbar(AggregateFunctionFactory &);
void registerAggregateFunctionIntervalLengthSum(AggregateFunctionFactory &);
class AggregateFunctionCombinatorFactory;
void registerAggregateFunctionCombinatorIf(AggregateFunctionCombinatorFactory &);
@ -66,8 +68,6 @@ void registerAggregateFunctionCombinatorDistinct(AggregateFunctionCombinatorFact
void registerWindowFunctions(AggregateFunctionFactory & factory);
void registerAggregateFunctionIntervalLengthSum(AggregateFunctionFactory &);
void registerAggregateFunctions()
{
{
@ -116,11 +116,11 @@ void registerAggregateFunctions()
registerAggregateFunctionWelchTTest(factory);
registerAggregateFunctionStudentTTest(factory);
registerAggregateFunctionSingleValueOrNull(factory);
registerAggregateFunctionIntervalLengthSum(factory);
registerAggregateFunctionExponentialMovingAverage(factory);
registerAggregateFunctionSparkbar(factory);
registerWindowFunctions(factory);
registerAggregateFunctionIntervalLengthSum(factory);
registerAggregateFunctionSparkbar(factory);
}
{

View File

@ -130,10 +130,16 @@ void Connection::connect(const ConnectionTimeouts & timeouts)
}
catch (Poco::TimeoutException & e)
{
/// disconnect() will reset the socket, get timeouts before.
const std::string & message = fmt::format("{} ({}, receive timeout {} ms, send timeout {} ms)",
e.displayText(), getDescription(),
socket->getReceiveTimeout().totalMilliseconds(),
socket->getSendTimeout().totalMilliseconds());
disconnect();
/// Add server address to exception. Also Exception will remember stack trace. It's a pity that more precise exception type is lost.
throw NetException(e.displayText() + " (" + getDescription() + ")", ErrorCodes::SOCKET_TIMEOUT);
throw NetException(message, ErrorCodes::SOCKET_TIMEOUT);
}
}

View File

@ -0,0 +1,114 @@
#pragma once
#include <cmath>
#include <limits>
namespace DB
{
/** https://en.wikipedia.org/wiki/Exponential_smoothing
*
* Exponentially smoothed average over time is weighted average with weight proportional to negative exponent of the time passed.
* For example, the last value is taken with weight 1/2, the value one second ago with weight 1/4, two seconds ago - 1/8, etc.
* It can be understood as an average over sliding window, but with different kernel.
*
* As an advantage, it is easy to update. Instead of collecting values and calculating a series of x1 / 2 + x2 / 4 + x3 / 8...
* just calculate x_old / 2 + x_new / 2.
*
* It is often used for resource usage metrics. For example, "load average" in Linux is exponentially smoothed moving average.
* We can use exponentially smoothed counters in query scheduler.
*/
struct ExponentiallySmoothedAverage
{
/// The sum. It contains the last value and all previous values scaled accordingly to the difference of their time to the reference time.
/// Older values are summed with exponentially smaller coefficients.
/// To obtain the average, you have to divide this value to the sum of all coefficients (see 'sumWeights').
double value = 0;
/// The point of reference. You can translate the value to a different point of reference (see 'remap').
/// You can imagine that the value exponentially decays over time.
/// But it is also meaningful to treat the whole counters as constants over time but in another non-linear coordinate system,
/// that inflates over time, while the counter itself does not change
/// (it continues to be the same physical quantity, but only changes its representation in the "usual" coordinate system).
/// Recap: the whole counter is one dimensional and it can be represented as a curve formed by two dependent coordinates in 2d plane,
/// the space can be represented by (value, time) coordinates, and the curves will be exponentially decaying over time,
/// alternatively the space can be represented by (exponentially_adjusted_value, time) and then the curves will be constant over time.
/// Also useful analogy is the exponential representation of a number: x = a * exp(b) = a * e (where e = exp(b))
/// a number x is represented by a curve in 2d plane that can be parametrized by coordinates (a, b) or (a, e).
double time = 0;
ExponentiallySmoothedAverage()
{
}
ExponentiallySmoothedAverage(double current_value, double current_time)
: value(current_value), time(current_time)
{
}
/// How much value decays after time_passed.
static double scale(double time_passed, double half_decay_time)
{
return exp2(-time_passed / half_decay_time);
}
/// Sum of weights of all values. Divide by it to get the average.
static double sumWeights(double half_decay_time)
{
double k = scale(1.0, half_decay_time);
return 1 / (1 - k);
}
/// Obtain the same counter in another point of reference.
ExponentiallySmoothedAverage remap(double current_time, double half_decay_time) const
{
return ExponentiallySmoothedAverage(value * scale(current_time - time, half_decay_time), current_time);
}
/// Merge two counters. It is done by moving to the same point of reference and summing the values.
static ExponentiallySmoothedAverage merge(const ExponentiallySmoothedAverage & a, const ExponentiallySmoothedAverage & b, double half_decay_time)
{
if (a.time > b.time)
return ExponentiallySmoothedAverage(a.value + b.remap(a.time, half_decay_time).value, a.time);
if (a.time < b.time)
return ExponentiallySmoothedAverage(b.value + a.remap(b.time, half_decay_time).value, b.time);
return ExponentiallySmoothedAverage(a.value + b.value, a.time);
}
void merge(const ExponentiallySmoothedAverage & other, double half_decay_time)
{
*this = merge(*this, other, half_decay_time);
}
void add(double new_value, double current_time, double half_decay_time)
{
merge(ExponentiallySmoothedAverage(new_value, current_time), half_decay_time);
}
/// Calculate the average from the sum.
double get(double half_decay_time) const
{
return value / sumWeights(half_decay_time);
}
double get(double current_time, double half_decay_time) const
{
return remap(current_time, half_decay_time).get(half_decay_time);
}
/// Compare two counters (by moving to the same point of reference and comparing sums).
/// You can store the counters in container and sort it without changing the stored values over time.
bool less(const ExponentiallySmoothedAverage & other, double half_decay_time) const
{
return remap(other.time, half_decay_time).value < other.value;
}
};
}

View File

@ -54,7 +54,9 @@ bool ReadBufferFromPocoSocket::nextImpl()
}
catch (const Poco::TimeoutException &)
{
throw NetException("Timeout exceeded while reading from socket (" + peer_address.toString() + ")", ErrorCodes::SOCKET_TIMEOUT);
throw NetException(fmt::format("Timeout exceeded while reading from socket ({}, {} ms)",
peer_address.toString(),
socket.impl()->getReceiveTimeout().totalMilliseconds()), ErrorCodes::SOCKET_TIMEOUT);
}
catch (const Poco::IOException & e)
{

View File

@ -57,7 +57,9 @@ void WriteBufferFromPocoSocket::nextImpl()
}
catch (const Poco::TimeoutException &)
{
throw NetException("Timeout exceeded while writing to socket (" + peer_address.toString() + ")", ErrorCodes::SOCKET_TIMEOUT);
throw NetException(fmt::format("Timeout exceeded while writing to socket ({}, {} ms)",
peer_address.toString(),
socket.impl()->getSendTimeout().totalMilliseconds()), ErrorCodes::SOCKET_TIMEOUT);
}
catch (const Poco::IOException & e)
{

View File

@ -24,7 +24,7 @@
FROM numbers_mt(200000000)
SETTINGS max_threads = 8
</fill_query>
<query>SELECT sum(x) FROM nullfloat32</query>
<query>SELECT sum(x::Nullable(Float64)) FROM nullfloat32</query>
<query short="1">SELECT sum(x) FROM nullfloat32</query>
<query short="1">SELECT sum(x::Nullable(Float64)) FROM nullfloat32</query>
<drop_query>DROP TABLE IF EXISTS nullfloat32</drop_query>
</test>

View File

@ -0,0 +1,130 @@
1 0 0.5
0 1 0.25
0 2 0.125
0 3 0.0625
0 4 0.03125
0 5 0.015625
0 6 0.0078125
0 7 0.00390625
0 8 0.001953125
0 9 0.0009765625
1 0 0.067
0 1 0.062
0 2 0.058
0 3 0.054
0 4 0.051
0 5 0.047
0 6 0.044
0 7 0.041
0 8 0.038
0 9 0.036
0 0 0
1 1 0.5
2 2 1.25
3 3 2.125
4 4 3.0625
5 5 4.03125
6 6 5.015625
7 7 6.0078125
8 8 7.00390625
9 9 8.001953125
1 0 0.067 ███▎
0 1 0.062 ███
0 2 0.058 ██▊
0 3 0.054 ██▋
0 4 0.051 ██▌
0 5 0.047 ██▎
0 6 0.044 ██▏
0 7 0.041 ██
0 8 0.038 █▊
0 9 0.036 █▋
0 10 0.033 █▋
0 11 0.031 █▌
0 12 0.029 █▍
0 13 0.027 █▎
0 14 0.025 █▎
0 15 0.024 █▏
0 16 0.022 █
0 17 0.021 █
0 18 0.019 ▊
0 19 0.018 ▊
0 20 0.017 ▋
0 21 0.016 ▋
0 22 0.015 ▋
0 23 0.014 ▋
0 24 0.013 ▋
1 25 0.079 ███▊
1 26 0.14 ███████
1 27 0.198 █████████▊
1 28 0.252 ████████████▌
1 29 0.302 ███████████████
1 30 0.349 █████████████████▍
1 31 0.392 ███████████████████▌
1 32 0.433 █████████████████████▋
1 33 0.471 ███████████████████████▌
1 34 0.506 █████████████████████████▎
1 35 0.539 ██████████████████████████▊
1 36 0.57 ████████████████████████████▌
1 37 0.599 █████████████████████████████▊
1 38 0.626 ███████████████████████████████▎
1 39 0.651 ████████████████████████████████▌
1 40 0.674 █████████████████████████████████▋
1 41 0.696 ██████████████████████████████████▋
1 42 0.716 ███████████████████████████████████▋
1 43 0.735 ████████████████████████████████████▋
1 44 0.753 █████████████████████████████████████▋
1 45 0.77 ██████████████████████████████████████▍
1 46 0.785 ███████████████████████████████████████▎
1 47 0.8 ███████████████████████████████████████▊
1 48 0.813 ████████████████████████████████████████▋
1 49 0.825 █████████████████████████████████████████▎
1 0 0.5 █████████████████████████
0 1 0.25 ████████████▌
0 2 0.125 ██████▎
0 3 0.062 ███
0 4 0.031 █▌
1 5 0.516 █████████████████████████▋
0 6 0.258 ████████████▊
0 7 0.129 ██████▍
0 8 0.064 ███▏
0 9 0.032 █▌
1 10 0.516 █████████████████████████▋
0 11 0.258 ████████████▊
0 12 0.129 ██████▍
0 13 0.065 ███▏
0 14 0.032 █▌
1 15 0.516 █████████████████████████▋
0 16 0.258 ████████████▊
0 17 0.129 ██████▍
0 18 0.065 ███▏
0 19 0.032 █▌
1 20 0.516 █████████████████████████▋
0 21 0.258 ████████████▊
0 22 0.129 ██████▍
0 23 0.065 ███▏
0 24 0.032 █▌
1 25 0.516 █████████████████████████▋
0 26 0.258 ████████████▊
0 27 0.129 ██████▍
0 28 0.065 ███▏
0 29 0.032 █▌
1 30 0.516 █████████████████████████▋
0 31 0.258 ████████████▊
0 32 0.129 ██████▍
0 33 0.065 ███▏
0 34 0.032 █▌
1 35 0.516 █████████████████████████▋
0 36 0.258 ████████████▊
0 37 0.129 ██████▍
0 38 0.065 ███▏
0 39 0.032 █▌
1 40 0.516 █████████████████████████▋
0 41 0.258 ████████████▊
0 42 0.129 ██████▍
0 43 0.065 ███▏
0 44 0.032 █▌
1 45 0.516 █████████████████████████▋
0 46 0.258 ████████████▊
0 47 0.129 ██████▍
0 48 0.065 ███▏
0 49 0.032 █▌

View File

@ -0,0 +1,32 @@
SELECT number = 0 AS value, number AS time, exponentialMovingAverage(1)(value, time) OVER (ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW) AS exp_smooth FROM numbers(10);
SELECT value, time, round(exp_smooth, 3) FROM (SELECT number = 0 AS value, number AS time, exponentialMovingAverage(10)(value, time) OVER (ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW) AS exp_smooth FROM numbers(10));
SELECT number AS value, number AS time, exponentialMovingAverage(1)(value, time) OVER (ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW) AS exp_smooth FROM numbers(10);
SELECT
value,
time,
round(exp_smooth, 3),
bar(exp_smooth, 0, 1, 50) AS bar
FROM
(
SELECT
(number = 0) OR (number >= 25) AS value,
number AS time,
exponentialMovingAverage(10)(value, time) OVER (ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW) AS exp_smooth
FROM numbers(50)
);
SELECT
value,
time,
round(exp_smooth, 3),
bar(exp_smooth, 0, 1, 50) AS bar
FROM
(
SELECT
(number % 5) = 0 AS value,
number AS time,
exponentialMovingAverage(1)(value, time) OVER (ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW) AS exp_smooth
FROM numbers(50)
);

View File

@ -0,0 +1,8 @@
0.0009765625
0.0009775171065493646
0.0009775171065493646
0.0009775171065493646
0.0009775171065493646
0.0009775171065493646
0.0009775171065493646
0.0009775171065493646

View File

@ -0,0 +1,9 @@
-- Check that it is deterministic
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(10);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(100);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(1000);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(10000);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(100000);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(1000000);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(10000000);
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM numbers_mt(100000000);

View File

@ -0,0 +1,5 @@
0.009775171065493644
0.009775171065493644
0.009775171065493644
0.009775171065493644
0.009775171065493644

View File

@ -0,0 +1,6 @@
-- Check that it is deterministic
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM remote('127.0.0.{1..10}', numbers_mt(1000));
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM remote('127.0.0.{1..10}', numbers_mt(10000));
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM remote('127.0.0.{1..10}', numbers_mt(100000));
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM remote('127.0.0.{1..10}', numbers_mt(1000000));
WITH number % 10 = 0 AS value, number AS time SELECT exponentialMovingAverage(1)(value, time) AS exp_smooth FROM remote('127.0.0.{1..10}', numbers_mt(10000000));