Merge branch 'master' of github.com:yandex/ClickHouse

This commit is contained in:
BayoNet 2019-08-21 10:53:28 +03:00
commit 1df058c64c
180 changed files with 4973 additions and 514 deletions

View File

@ -9,6 +9,9 @@
* `RENAME` queries now work with all storages. [#5953](https://github.com/yandex/ClickHouse/pull/5953) ([Ivan](https://github.com/abyss7))
* Now client receive logs from server with any desired level by setting `send_logs_level` regardless to the log level specified in server settings. [#5964](https://github.com/yandex/ClickHouse/pull/5964) ([Nikita Mikhaylov](https://github.com/nikitamikhaylov))
### Backward Incompatible Change
* The setting `input_format_defaults_for_omitted_fields` is enabled by default. Inserts in Distibuted tables need this setting to be the same on cluster (you need to set it before rolling update). It enables calculation of complex default expressions for omitted fields in `JSONEachRow` and `CSV*` formats. It should be the expected behaviour but may lead to negligible performance difference. [#6043](https://github.com/yandex/ClickHouse/pull/6043) ([Artem Zuikov](https://github.com/4ertus2)), [#5625](https://github.com/yandex/ClickHouse/pull/5625) ([akuzm](https://github.com/akuzm))
### Experimental features
* New query processing pipeline. Use `experimental_use_processors=1` option to enable it. Use for your own trouble. [#4914](https://github.com/yandex/ClickHouse/pull/4914) ([Nikolai Kochetov](https://github.com/KochetovNicolai))
@ -18,7 +21,6 @@
* Fixed overestimation of `max_rows_to_read` if the setting `merge_tree_uniform_read_distribution` is set to 0. [#6019](https://github.com/yandex/ClickHouse/pull/6019) ([alexey-milovidov](https://github.com/alexey-milovidov))
### Improvement
* The setting `input_format_defaults_for_omitted_fields` is enabled by default. It enables calculation of complex default expressions for omitted fields in `JSONEachRow` and `CSV*` formats. It should be the expected behaviour but may lead to negligible performance difference or subtle incompatibilities. [#6043](https://github.com/yandex/ClickHouse/pull/6043) ([Artem Zuikov](https://github.com/4ertus2)), [#5625](https://github.com/yandex/ClickHouse/pull/5625) ([akuzm](https://github.com/akuzm))
* Throws an exception if `config.d` file doesn't have the corresponding root element as the config file [#6123](https://github.com/yandex/ClickHouse/pull/6123) ([dimarub2000](https://github.com/dimarub2000))
### Performance Improvement

View File

@ -2,7 +2,7 @@ option (ENABLE_PARQUET "Enable parquet" ON)
if (ENABLE_PARQUET)
if (NOT OS_FREEBSD) # Freebsd: ../contrib/arrow/cpp/src/arrow/util/bit-util.h:27:10: fatal error: endian.h: No such file or directory
if (NOT OS_FREEBSD AND NOT APPLE) # Freebsd: ../contrib/arrow/cpp/src/arrow/util/bit-util.h:27:10: fatal error: endian.h: No such file or directory
option(USE_INTERNAL_PARQUET_LIBRARY "Set to FALSE to use system parquet library instead of bundled" ${NOT_UNBUNDLED})
endif()

View File

@ -1932,15 +1932,13 @@ protected:
TaskTable & task_table = task_shard.task_table;
String query;
{
WriteBufferFromOwnString wb;
wb << "SELECT 1"
<< " FROM "<< getQuotedTable(task_shard.table_read_shard)
<< " WHERE " << queryToString(task_table.engine_push_partition_key_ast) << " = " << partition_quoted_name
<< " LIMIT 1";
query = wb.str();
}
std::string query = "SELECT 1 FROM " + getQuotedTable(task_shard.table_read_shard)
+ " WHERE (" + queryToString(task_table.engine_push_partition_key_ast) + " = (" + partition_quoted_name + " AS partition_key))";
if (!task_table.where_condition_str.empty())
query += " AND (" + task_table.where_condition_str + ")";
query += " LIMIT 1";
LOG_DEBUG(log, "Checking shard " << task_shard.getDescription() << " for partition "
<< partition_quoted_name << " existence, executing query: " << query);

View File

@ -1,3 +1,5 @@
#ifdef __ELF__
/*
* Copyright 2012-present Facebook, Inc.
*
@ -1031,3 +1033,5 @@ bool Dwarf::LineNumberVM::findAddress(uintptr_t target, Path & file, uint64_t &
}
}
#endif

View File

@ -1,5 +1,7 @@
#pragma once
#ifdef __ELF__
/*
* Copyright 2012-present Facebook, Inc.
*
@ -285,3 +287,5 @@ private:
};
}
#endif

View File

@ -1,3 +1,5 @@
#ifdef __ELF__
#include <Common/Elf.h>
#include <Common/Exception.h>
@ -128,3 +130,5 @@ size_t Elf::Section::size() const
}
}
#endif

View File

@ -1,5 +1,7 @@
#pragma once
#ifdef __ELF__
#include <IO/MMapReadBufferFromFile.h>
#include <string>
@ -61,3 +63,5 @@ private:
};
}
#endif

View File

@ -443,6 +443,7 @@ namespace ErrorCodes
extern const int INSECURE_PATH = 466;
extern const int CANNOT_PARSE_BOOL = 467;
extern const int CANNOT_PTHREAD_ATTR = 468;
extern const int QUERY_IS_NOT_SUPPORTED_IN_LIVE_VIEW = 469;
extern const int KEEPER_EXCEPTION = 999;
extern const int POCO_EXCEPTION = 1000;

View File

@ -250,6 +250,7 @@ static void toStringEveryLineImpl(const StackTrace::Frames & frames, size_t offs
if (size == 0)
return callback("<Empty trace>");
#ifdef __ELF__
const DB::SymbolIndex & symbol_index = DB::SymbolIndex::instance();
std::unordered_map<std::string, DB::Dwarf> dwarfs;
@ -290,6 +291,18 @@ static void toStringEveryLineImpl(const StackTrace::Frames & frames, size_t offs
callback(out.str());
out.str({});
}
#else
std::stringstream out;
for (size_t i = offset; i < size; ++i)
{
const void * addr = frames[i];
out << i << ". " << addr;
callback(out.str());
out.str({});
}
#endif
}
static std::string toStringImpl(const StackTrace::Frames & frames, size_t offset, size_t size)

View File

@ -1,3 +1,5 @@
#ifdef __ELF__
#include <Common/SymbolIndex.h>
#include <algorithm>
@ -316,3 +318,5 @@ const SymbolIndex::Object * SymbolIndex::findObject(const void * address) const
}
}
#endif

View File

@ -1,5 +1,7 @@
#pragma once
#ifdef __ELF__
#include <vector>
#include <string>
#include <ext/singleton.h>
@ -53,3 +55,5 @@ private:
};
}
#endif

View File

@ -1,7 +1,8 @@
#pragma once
#include <tuple>
#include <sstream>
#include <iomanip>
#include <city.h>
#include <Core/Types.h>
@ -33,6 +34,13 @@ struct UInt128
auto tuple() const { return std::tie(high, low); }
String toHexString() const
{
std::ostringstream os;
os << std::setw(16) << std::setfill('0') << std::hex << high << low;
return String(os.str());
}
bool inline operator== (const UInt128 rhs) const { return tuple() == rhs.tuple(); }
bool inline operator!= (const UInt128 rhs) const { return tuple() != rhs.tuple(); }
bool inline operator< (const UInt128 rhs) const { return tuple() < rhs.tuple(); }

View File

@ -16,6 +16,7 @@ using namespace DB;
int main(int argc, char ** argv)
{
#ifdef __ELF__
if (argc < 2)
{
std::cerr << "Usage: ./symbol_index address\n";
@ -53,6 +54,12 @@ int main(int argc, char ** argv)
std::cerr << "\n";
std::cerr << StackTrace().toString() << "\n";
#else
(void)argc;
(void)argv;
std::cerr << "This test does not make sense for non-ELF objects.\n";
#endif
return 0;
}

View File

@ -144,7 +144,8 @@ private:
using Blocks = std::vector<Block>;
using BlocksList = std::list<Block>;
using BlocksPtr = std::shared_ptr<Blocks>;
using BlocksPtrs = std::shared_ptr<std::vector<BlocksPtr>>;
/// Compare number of columns, data types, column types, column names, and values of constant columns.
bool blocksHaveEqualStructure(const Block & lhs, const Block & rhs);

View File

@ -32,7 +32,11 @@
*/
#define DEFAULT_MERGE_BLOCK_SIZE 8192
#define DEFAULT_TEMPORARY_LIVE_VIEW_TIMEOUT_SEC 5
#define DEFAULT_TEMPORARY_LIVE_CHANNEL_TIMEOUT_SEC 15
#define DEFAULT_ALTER_LIVE_CHANNEL_WAIT_MS 10000
#define SHOW_CHARS_ON_SYNTAX_ERROR ptrdiff_t(160)
#define DEFAULT_LIVE_VIEW_HEARTBEAT_INTERVAL_SEC 15
#define DBMS_DEFAULT_DISTRIBUTED_CONNECTIONS_POOL_SIZE 1024
#define DBMS_CONNECTION_POOL_WITH_FAILOVER_DEFAULT_MAX_TRIES 3
/// each period reduces the error counter by 2 times

View File

@ -345,6 +345,12 @@ struct Settings : public SettingsCollection<Settings>
/** Obsolete settings that do nothing but left for compatibility reasons. Remove each one after half a year of obsolescence. */ \
\
M(SettingBool, allow_experimental_low_cardinality_type, true, "Obsolete setting, does nothing. Will be removed after 2019-08-13") \
\
M(SettingSeconds, live_view_heartbeat_interval, DEFAULT_LIVE_VIEW_HEARTBEAT_INTERVAL_SEC, "The heartbeat interval in seconds to indicate live query is alive.") \
M(SettingSeconds, temporary_live_view_timeout, DEFAULT_TEMPORARY_LIVE_VIEW_TIMEOUT_SEC, "Timeout after which temporary live view is deleted.") \
M(SettingSeconds, temporary_live_channel_timeout, DEFAULT_TEMPORARY_LIVE_CHANNEL_TIMEOUT_SEC, "Timeout after which temporary live channel is deleted.") \
M(SettingMilliseconds, alter_channel_wait_ms, DEFAULT_ALTER_LIVE_CHANNEL_WAIT_MS, "The wait time for alter channel request.") \
M(SettingUInt64, max_live_view_insert_blocks_before_refresh, 64, "Limit maximum number of inserted blocks after which mergeable blocks are dropped and query is re-executed.") \
DECLARE_SETTINGS_COLLECTION(LIST_OF_SETTINGS)

View File

@ -0,0 +1,51 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <DataStreams/IBlockInputStream.h>
namespace DB
{
/** A stream of blocks from a shared vector of blocks
*/
class BlocksBlockInputStream : public IBlockInputStream
{
public:
/// Acquires shared ownership of the blocks vector
BlocksBlockInputStream(const std::shared_ptr<BlocksPtr> & blocks_ptr_, Block header_)
: blocks(*blocks_ptr_), it((*blocks_ptr_)->begin()), end((*blocks_ptr_)->end()), header(std::move(header_)) {}
String getName() const override { return "Blocks"; }
Block getHeader() const override { return header; }
protected:
Block readImpl() override
{
if (it == end)
return Block();
Block res = *it;
++it;
return res;
}
private:
BlocksPtr blocks;
Blocks::iterator it;
const Blocks::iterator end;
Block header;
};
}

View File

@ -0,0 +1,222 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <limits>
#include <Common/ConcurrentBoundedQueue.h>
#include <Poco/Condition.h>
#include <DataStreams/OneBlockInputStream.h>
#include <DataStreams/IBlockInputStream.h>
#include <Storages/StorageLiveView.h>
namespace DB
{
/** Implements LIVE VIEW table WATCH input stream.
* Keeps stream alive by outputing blocks with no rows
* based on period specified by the heartbeat interval.
*/
class LiveViewBlockInputStream : public IBlockInputStream
{
using NonBlockingResult = std::pair<Block, bool>;
public:
~LiveViewBlockInputStream() override
{
/// Start storage no users thread
/// if we are the last active user
if (!storage->is_dropped && blocks_ptr.use_count() < 3)
storage->startNoUsersThread(temporary_live_view_timeout_sec);
}
LiveViewBlockInputStream(std::shared_ptr<StorageLiveView> storage_,
std::shared_ptr<BlocksPtr> blocks_ptr_,
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr_,
std::shared_ptr<bool> active_ptr_,
const bool has_limit_, const UInt64 limit_,
const UInt64 heartbeat_interval_sec_,
const UInt64 temporary_live_view_timeout_sec_)
: storage(std::move(storage_)), blocks_ptr(std::move(blocks_ptr_)), blocks_metadata_ptr(std::move(blocks_metadata_ptr_)), active_ptr(std::move(active_ptr_)), has_limit(has_limit_), limit(limit_), heartbeat_interval_usec(heartbeat_interval_sec_ * 1000000), temporary_live_view_timeout_sec(temporary_live_view_timeout_sec_)
{
/// grab active pointer
active = active_ptr.lock();
}
String getName() const override { return "LiveViewBlockInputStream"; }
void cancel(bool kill) override
{
if (isCancelled() || storage->is_dropped)
return;
IBlockInputStream::cancel(kill);
Poco::FastMutex::ScopedLock lock(storage->mutex);
storage->condition.broadcast();
}
Block getHeader() const override { return storage->getHeader(); }
void refresh()
{
if (active && blocks && it == end)
it = blocks->begin();
}
void suspend()
{
active.reset();
}
void resume()
{
active = active_ptr.lock();
{
if (!blocks || blocks.get() != (*blocks_ptr).get())
blocks = (*blocks_ptr);
}
it = blocks->begin();
begin = blocks->begin();
end = blocks->end();
}
NonBlockingResult tryRead()
{
return tryRead_(false);
}
protected:
Block readImpl() override
{
/// try reading
return tryRead_(true).first;
}
/** tryRead method attempts to read a block in either blocking
* or non-blocking mode. If blocking is set to false
* then method return empty block with flag set to false
* to indicate that method would block to get the next block.
*/
NonBlockingResult tryRead_(bool blocking)
{
Block res;
if (has_limit && num_updates == static_cast<Int64>(limit))
{
return { Block(), true };
}
/// If blocks were never assigned get blocks
if (!blocks)
{
Poco::FastMutex::ScopedLock lock(storage->mutex);
if (!active)
return { Block(), false };
blocks = (*blocks_ptr);
it = blocks->begin();
begin = blocks->begin();
end = blocks->end();
}
if (isCancelled() || storage->is_dropped)
{
return { Block(), true };
}
if (it == end)
{
{
Poco::FastMutex::ScopedLock lock(storage->mutex);
if (!active)
return { Block(), false };
/// If we are done iterating over our blocks
/// and there are new blocks availble then get them
if (blocks.get() != (*blocks_ptr).get())
{
blocks = (*blocks_ptr);
it = blocks->begin();
begin = blocks->begin();
end = blocks->end();
}
/// No new blocks available wait for new ones
else
{
if (!blocking)
{
return { Block(), false };
}
if (!end_of_blocks)
{
end_of_blocks = true;
return { getHeader(), true };
}
while (true)
{
UInt64 timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
bool signaled = storage->condition.tryWait(storage->mutex, std::max(static_cast<UInt64>(0), heartbeat_interval_usec - (timestamp_usec - last_event_timestamp_usec)) / 1000);
if (isCancelled() || storage->is_dropped)
{
return { Block(), true };
}
if (signaled)
{
break;
}
else
{
// heartbeat
last_event_timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
return { getHeader(), true };
}
}
}
}
return tryRead_(blocking);
}
res = *it;
++it;
if (it == end)
{
end_of_blocks = false;
num_updates += 1;
}
last_event_timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
return { res, true };
}
private:
std::shared_ptr<StorageLiveView> storage;
std::shared_ptr<BlocksPtr> blocks_ptr;
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr;
std::weak_ptr<bool> active_ptr;
std::shared_ptr<bool> active;
BlocksPtr blocks;
BlocksMetadataPtr blocks_metadata;
Blocks::iterator it;
Blocks::iterator end;
Blocks::iterator begin;
const bool has_limit;
const UInt64 limit;
Int64 num_updates = -1;
bool end_of_blocks = false;
UInt64 heartbeat_interval_usec;
UInt64 temporary_live_view_timeout_sec;
UInt64 last_event_timestamp_usec = 0;
Poco::Timestamp timestamp;
};
}

View File

@ -0,0 +1,243 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <limits>
#include <Common/ConcurrentBoundedQueue.h>
#include <Poco/Condition.h>
#include <DataTypes/DataTypesNumber.h>
#include <DataTypes/DataTypeString.h>
#include <Columns/ColumnString.h>
#include <Columns/ColumnsNumber.h>
#include <DataStreams/OneBlockInputStream.h>
#include <DataStreams/IBlockInputStream.h>
#include <Storages/StorageLiveView.h>
namespace DB
{
/** Implements LIVE VIEW table WATCH EVENTS input stream.
* Keeps stream alive by outputing blocks with no rows
* based on period specified by the heartbeat interval.
*/
class LiveViewEventsBlockInputStream : public IBlockInputStream
{
using NonBlockingResult = std::pair<Block, bool>;
public:
~LiveViewEventsBlockInputStream() override
{
/// Start storage no users thread
/// if we are the last active user
if (!storage->is_dropped && blocks_ptr.use_count() < 3)
storage->startNoUsersThread(temporary_live_view_timeout_sec);
}
/// length default -2 because we want LIMIT to specify number of updates so that LIMIT 1 waits for 1 update
/// and LIMIT 0 just returns data without waiting for any updates
LiveViewEventsBlockInputStream(std::shared_ptr<StorageLiveView> storage_,
std::shared_ptr<BlocksPtr> blocks_ptr_,
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr_,
std::shared_ptr<bool> active_ptr_,
const bool has_limit_, const UInt64 limit_,
const UInt64 heartbeat_interval_sec_,
const UInt64 temporary_live_view_timeout_sec_)
: storage(std::move(storage_)), blocks_ptr(std::move(blocks_ptr_)), blocks_metadata_ptr(std::move(blocks_metadata_ptr_)), active_ptr(std::move(active_ptr_)), has_limit(has_limit_), limit(limit_), heartbeat_interval_usec(heartbeat_interval_sec_ * 1000000), temporary_live_view_timeout_sec(temporary_live_view_timeout_sec_)
{
/// grab active pointer
active = active_ptr.lock();
}
String getName() const override { return "LiveViewEventsBlockInputStream"; }
void cancel(bool kill) override
{
if (isCancelled() || storage->is_dropped)
return;
IBlockInputStream::cancel(kill);
Poco::FastMutex::ScopedLock lock(storage->mutex);
storage->condition.broadcast();
}
Block getHeader() const override
{
return {ColumnWithTypeAndName(ColumnUInt64::create(), std::make_shared<DataTypeUInt64>(), "version")};
}
void refresh()
{
if (active && blocks && it == end)
it = blocks->begin();
}
void suspend()
{
active.reset();
}
void resume()
{
active = active_ptr.lock();
{
if (!blocks || blocks.get() != (*blocks_ptr).get())
{
blocks = (*blocks_ptr);
blocks_metadata = (*blocks_metadata_ptr);
}
}
it = blocks->begin();
begin = blocks->begin();
end = blocks->end();
}
NonBlockingResult tryRead()
{
return tryRead_(false);
}
Block getEventBlock()
{
Block res{
ColumnWithTypeAndName(
DataTypeUInt64().createColumnConst(1, blocks_metadata->version)->convertToFullColumnIfConst(),
std::make_shared<DataTypeUInt64>(),
"version")
};
return res;
}
protected:
Block readImpl() override
{
/// try reading
return tryRead_(true).first;
}
/** tryRead method attempts to read a block in either blocking
* or non-blocking mode. If blocking is set to false
* then method return empty block with flag set to false
* to indicate that method would block to get the next block.
*/
NonBlockingResult tryRead_(bool blocking)
{
if (has_limit && num_updates == static_cast<Int64>(limit))
{
return { Block(), true };
}
/// If blocks were never assigned get blocks
if (!blocks)
{
Poco::FastMutex::ScopedLock lock(storage->mutex);
if (!active)
return { Block(), false };
blocks = (*blocks_ptr);
blocks_metadata = (*blocks_metadata_ptr);
it = blocks->begin();
begin = blocks->begin();
end = blocks->end();
}
if (isCancelled() || storage->is_dropped)
{
return { Block(), true };
}
if (it == end)
{
{
Poco::FastMutex::ScopedLock lock(storage->mutex);
if (!active)
return { Block(), false };
/// If we are done iterating over our blocks
/// and there are new blocks availble then get them
if (blocks.get() != (*blocks_ptr).get())
{
blocks = (*blocks_ptr);
blocks_metadata = (*blocks_metadata_ptr);
it = blocks->begin();
begin = blocks->begin();
end = blocks->end();
}
/// No new blocks available wait for new ones
else
{
if (!blocking)
{
return { Block(), false };
}
if (!end_of_blocks)
{
end_of_blocks = true;
return { getHeader(), true };
}
while (true)
{
UInt64 timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
bool signaled = storage->condition.tryWait(storage->mutex, std::max(static_cast<UInt64>(0), heartbeat_interval_usec - (timestamp_usec - last_event_timestamp_usec)) / 1000);
if (isCancelled() || storage->is_dropped)
{
return { Block(), true };
}
if (signaled)
{
break;
}
else
{
// repeat the event block as a heartbeat
last_event_timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
return { getHeader(), true };
}
}
}
}
return tryRead_(blocking);
}
// move right to the end
it = end;
if (it == end)
{
end_of_blocks = false;
num_updates += 1;
}
last_event_timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
return { getEventBlock(), true };
}
private:
std::shared_ptr<StorageLiveView> storage;
std::shared_ptr<BlocksPtr> blocks_ptr;
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr;
std::weak_ptr<bool> active_ptr;
std::shared_ptr<bool> active;
BlocksPtr blocks;
BlocksMetadataPtr blocks_metadata;
Blocks::iterator it;
Blocks::iterator end;
Blocks::iterator begin;
const bool has_limit;
const UInt64 limit;
Int64 num_updates = -1;
bool end_of_blocks = false;
UInt64 heartbeat_interval_usec;
UInt64 temporary_live_view_timeout_sec;
UInt64 last_event_timestamp_usec = 0;
Poco::Timestamp timestamp;
};
}

View File

@ -12,6 +12,7 @@
#include <Common/ThreadPool.h>
#include <Storages/MergeTree/ReplicatedMergeTreeBlockOutputStream.h>
#include <Storages/StorageValues.h>
#include <Storages/StorageLiveView.h>
namespace DB
{
@ -47,17 +48,30 @@ PushingToViewsBlockOutputStream::PushingToViewsBlockOutputStream(
for (const auto & database_table : dependencies)
{
auto dependent_table = context.getTable(database_table.first, database_table.second);
auto & materialized_view = dynamic_cast<const StorageMaterializedView &>(*dependent_table);
StoragePtr inner_table = materialized_view.getTargetTable();
auto query = materialized_view.getInnerQuery();
std::unique_ptr<ASTInsertQuery> insert = std::make_unique<ASTInsertQuery>();
insert->database = inner_table->getDatabaseName();
insert->table = inner_table->getTableName();
ASTPtr insert_query_ptr(insert.release());
InterpreterInsertQuery interpreter(insert_query_ptr, *views_context);
BlockIO io = interpreter.execute();
views.emplace_back(ViewInfo{query, database_table.first, database_table.second, io.out});
ASTPtr query;
BlockOutputStreamPtr out;
if (auto * materialized_view = dynamic_cast<const StorageMaterializedView *>(dependent_table.get()))
{
StoragePtr inner_table = materialized_view->getTargetTable();
query = materialized_view->getInnerQuery();
std::unique_ptr<ASTInsertQuery> insert = std::make_unique<ASTInsertQuery>();
insert->database = inner_table->getDatabaseName();
insert->table = inner_table->getTableName();
ASTPtr insert_query_ptr(insert.release());
InterpreterInsertQuery interpreter(insert_query_ptr, *views_context);
BlockIO io = interpreter.execute();
out = io.out;
}
else if (dynamic_cast<const StorageLiveView *>(dependent_table.get()))
out = std::make_shared<PushingToViewsBlockOutputStream>(
database_table.first, database_table.second, dependent_table, *views_context, ASTPtr(), true);
else
out = std::make_shared<PushingToViewsBlockOutputStream>(
database_table.first, database_table.second, dependent_table, *views_context, ASTPtr());
views.emplace_back(ViewInfo{std::move(query), database_table.first, database_table.second, std::move(out)});
}
}
@ -90,10 +104,18 @@ void PushingToViewsBlockOutputStream::write(const Block & block)
*/
Nested::validateArraySizes(block);
if (output)
/// TODO: to support virtual and alias columns inside MVs, we should return here the inserted block extended
/// with additional columns directly from storage and pass it to MVs instead of raw block.
output->write(block);
if (auto * live_view = dynamic_cast<StorageLiveView *>(storage.get()))
{
BlockOutputStreamPtr output_ = std::make_shared<LiveViewBlockOutputStream>(*live_view);
StorageLiveView::writeIntoLiveView(*live_view, block, context, output_);
}
else
{
if (output)
/// TODO: to support virtual and alias columns inside MVs, we should return here the inserted block extended
/// with additional columns directly from storage and pass it to MVs instead of raw block.
output->write(block);
}
/// Don't process materialized views if this block is duplicate
if (replicated_output && replicated_output->lastBlockIsDuplicate())
@ -180,20 +202,29 @@ void PushingToViewsBlockOutputStream::process(const Block & block, size_t view_n
try
{
/// We create a table with the same name as original table and the same alias columns,
/// but it will contain single block (that is INSERT-ed into main table).
/// InterpreterSelectQuery will do processing of alias columns.
Context local_context = *views_context;
local_context.addViewSource(StorageValues::create(storage->getDatabaseName(), storage->getTableName(), storage->getColumns(), block));
InterpreterSelectQuery select(view.query, local_context, SelectQueryOptions());
BlockInputStreamPtr in;
BlockInputStreamPtr in = std::make_shared<MaterializingBlockInputStream>(select.execute().in);
/// Squashing is needed here because the materialized view query can generate a lot of blocks
/// even when only one block is inserted into the parent table (e.g. if the query is a GROUP BY
/// and two-level aggregation is triggered).
in = std::make_shared<SquashingBlockInputStream>(
in, context.getSettingsRef().min_insert_block_size_rows, context.getSettingsRef().min_insert_block_size_bytes);
in = std::make_shared<ConvertingBlockInputStream>(context, in, view.out->getHeader(), ConvertingBlockInputStream::MatchColumnsMode::Name);
if (view.query)
{
/// We create a table with the same name as original table and the same alias columns,
/// but it will contain single block (that is INSERT-ed into main table).
/// InterpreterSelectQuery will do processing of alias columns.
Context local_context = *views_context;
local_context.addViewSource(
StorageValues::create(storage->getDatabaseName(), storage->getTableName(), storage->getColumns(),
block));
InterpreterSelectQuery select(view.query, local_context, SelectQueryOptions());
in = std::make_shared<MaterializingBlockInputStream>(select.execute().in);
/// Squashing is needed here because the materialized view query can generate a lot of blocks
/// even when only one block is inserted into the parent table (e.g. if the query is a GROUP BY
/// and two-level aggregation is triggered).
in = std::make_shared<SquashingBlockInputStream>(
in, context.getSettingsRef().min_insert_block_size_rows, context.getSettingsRef().min_insert_block_size_bytes);
in = std::make_shared<ConvertingBlockInputStream>(context, in, view.out->getHeader(), ConvertingBlockInputStream::MatchColumnsMode::Name);
}
else
in = std::make_shared<OneBlockInputStream>(block);
in->readPrefix();

View File

@ -5,7 +5,7 @@
#include <DataStreams/OneBlockInputStream.h>
#include <DataStreams/MaterializingBlockInputStream.h>
#include <Storages/StorageMaterializedView.h>
#include <Storages/StorageLiveView.h>
namespace DB
{

View File

@ -164,7 +164,18 @@ static Block adaptBlockStructure(const Block & block, const Block & header, cons
res.info = block.info;
for (const auto & elem : header)
res.insert({ castColumn(block.getByName(elem.name), elem.type, context), elem.type, elem.name });
{
ColumnPtr column;
if (elem.column && isColumnConst(*elem.column))
/// TODO: check that column from block contains the same value.
/// TODO: serialize const columns.
column = elem.column->cloneResized(block.rows());
else
column = castColumn(block.getByName(elem.name), elem.type, context);
res.insert({column, elem.type, elem.name});
}
return res;
}

View File

@ -28,6 +28,8 @@ void copyDataImpl(IBlockInputStream & from, IBlockOutputStream & to, TCancelCall
break;
to.write(block);
if (!block.rows())
to.flush();
progress(block);
}

View File

@ -41,7 +41,7 @@ String getTableDefinitionFromCreateQuery(const ASTPtr & query)
create.replace_view = false;
/// For views it is necessary to save the SELECT query itself, for the rest - on the contrary
if (!create.is_view && !create.is_materialized_view)
if (!create.is_view && !create.is_materialized_view && !create.is_live_view)
create.select = nullptr;
create.format = nullptr;

View File

@ -252,6 +252,7 @@ void registerOutputFormatProcessorPrettySpace(FormatFactory & factory);
void registerOutputFormatProcessorVertical(FormatFactory & factory);
void registerOutputFormatProcessorJSON(FormatFactory & factory);
void registerOutputFormatProcessorJSONCompact(FormatFactory & factory);
void registerOutputFormatProcessorJSONEachRowWithProgress(FormatFactory & factory);
void registerOutputFormatProcessorXML(FormatFactory & factory);
void registerOutputFormatProcessorODBCDriver(FormatFactory & factory);
void registerOutputFormatProcessorODBCDriver2(FormatFactory & factory);
@ -268,6 +269,8 @@ FormatFactory::FormatFactory()
registerInputFormatTabSeparated(*this);
registerInputFormatCSV(*this);
registerOutputFormatProcessorJSONEachRowWithProgress(*this);
registerInputFormatProcessorNative(*this);
registerOutputFormatProcessorNative(*this);
registerInputFormatProcessorRowBinary(*this);

View File

@ -159,6 +159,13 @@ public:
*/
virtual bool isSuitableForConstantFolding() const { return true; }
/** Some functions like ignore(...) or toTypeName(...) always return constant result which doesn't depend on arguments.
* In this case we can calculate result and assume that it's constant in stream header.
* There is no need to implement function if it has zero arguments.
* Must return ColumnConst with single row or nullptr.
*/
virtual ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block & /*block*/, const ColumnNumbers & /*arguments*/) const { return nullptr; }
/** Function is called "injective" if it returns different result for different values of arguments.
* Example: hex, negate, tuple...
*
@ -456,6 +463,10 @@ public:
}
bool isSuitableForConstantFolding() const override { return function->isSuitableForConstantFolding(); }
ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block & block, const ColumnNumbers & arguments_) const override
{
return function->getResultIfAlwaysReturnsConstantAndHasArguments(block, arguments_);
}
bool isInjective(const Block & sample_block) override { return function->isInjective(sample_block); }

View File

@ -1,3 +1,5 @@
#ifdef __ELF__
#include <Common/Elf.h>
#include <Common/Dwarf.h>
#include <Common/SymbolIndex.h>
@ -149,3 +151,5 @@ void registerFunctionAddressToLine(FunctionFactory & factory)
}
}
#endif

View File

@ -1,3 +1,5 @@
#ifdef __ELF__
#include <Common/SymbolIndex.h>
#include <Columns/ColumnString.h>
#include <Columns/ColumnsNumber.h>
@ -92,3 +94,5 @@ void registerFunctionAddressToSymbol(FunctionFactory & factory)
}
}
#endif

View File

@ -72,7 +72,7 @@ public:
offsets.push_back(offset);
}
block.getByPosition(result).column = ColumnArray::create(col_value->replicate(offsets), std::move(offsets_col));
block.getByPosition(result).column = ColumnArray::create(col_value->replicate(offsets)->convertToFullColumnIfConst(), std::move(offsets_col));
}
};

View File

@ -37,6 +37,12 @@ public:
const IDataType & type = *block.getByPosition(arguments[0]).type;
block.getByPosition(result).column = type.createColumnConst(input_rows_count, type.getDefault());
}
ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block & block, const ColumnNumbers & arguments) const override
{
const IDataType & type = *block.getByPosition(arguments[0]).type;
return type.createColumnConst(1, type.getDefault());
}
};

View File

@ -49,11 +49,16 @@ public:
}
void executeImpl(Block & block, const ColumnNumbers & arguments, size_t result, size_t input_rows_count) override
{
block.getByPosition(result).column = getResultIfAlwaysReturnsConstantAndHasArguments(block, arguments)->cloneResized(input_rows_count);
}
ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block & block, const ColumnNumbers & arguments) const override
{
if (auto type8 = checkAndGetDataType<DataTypeEnum8>(block.getByPosition(arguments[0]).type.get()))
block.getByPosition(result).column = DataTypeUInt8().createColumnConst(input_rows_count, type8->getValues().size());
return DataTypeUInt8().createColumnConst(1, type8->getValues().size());
else if (auto type16 = checkAndGetDataType<DataTypeEnum16>(block.getByPosition(arguments[0]).type.get()))
block.getByPosition(result).column = DataTypeUInt16().createColumnConst(input_rows_count, type16->getValues().size());
return DataTypeUInt16().createColumnConst(1, type16->getValues().size());
else
throw Exception("The argument for function " + getName() + " must be Enum", ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
}

View File

@ -42,6 +42,11 @@ public:
{
block.getByPosition(result).column = DataTypeUInt8().createColumnConst(input_rows_count, 0u);
}
ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block &, const ColumnNumbers &) const override
{
return DataTypeUInt8().createColumnConst(1, 0u);
}
};

View File

@ -1,6 +1,7 @@
#include <Functions/IFunction.h>
#include <Functions/FunctionFactory.h>
#include <DataTypes/DataTypesNumber.h>
#include <Columns/ColumnsNumber.h>
namespace DB
@ -38,7 +39,11 @@ namespace DB
void executeImpl(Block & block, const ColumnNumbers &, size_t result, size_t input_rows_count) override
{
block.getByPosition(result).column = DataTypeUInt8().createColumnConst(input_rows_count, UInt64(0));
/// This function is mainly used in query analysis instead of "in" functions
/// in the case when only header is needed and set for in is not calculated.
/// Because of that function must return the same column type as "in" function, which is ColumnUInt8.
auto res = ColumnUInt8::create(input_rows_count, 0);
block.getByPosition(result).column = std::move(res);
}
};

View File

@ -75,6 +75,8 @@ public:
void executeImpl(Block & block, const ColumnNumbers & arguments, size_t result, size_t /*input_rows_count*/) override
{
/// NOTE: after updating this code, check that FunctionIgnoreExceptNull returns the same type of column.
/// Second argument must be ColumnSet.
ColumnPtr column_set_ptr = block.getByPosition(arguments[1]).column;
const ColumnSet * column_set = typeid_cast<const ColumnSet *>(&*column_set_ptr);

View File

@ -43,18 +43,17 @@ public:
void executeImpl(Block & block, const ColumnNumbers & arguments, size_t result, size_t input_rows_count) override
{
/// nullIf(col1, col2) == if(col1 != col2, col1, NULL)
/// nullIf(col1, col2) == if(col1 = col2, NULL, 1)
Block temp_block = block;
size_t res_pos = temp_block.columns();
temp_block.insert({nullptr, std::make_shared<DataTypeUInt8>(), ""});
auto equals_func = FunctionFactory::instance().get("equals", context)->build(
{temp_block.getByPosition(arguments[0]), temp_block.getByPosition(arguments[1])});
{
auto equals_func = FunctionFactory::instance().get("notEquals", context)->build(
{temp_block.getByPosition(arguments[0]), temp_block.getByPosition(arguments[1])});
equals_func->execute(temp_block, {arguments[0], arguments[1]}, res_pos, input_rows_count);
}
size_t equals_res_pos = temp_block.columns();
temp_block.insert({nullptr, equals_func->getReturnType(), ""});
equals_func->execute(temp_block, {arguments[0], arguments[1]}, equals_res_pos, input_rows_count);
/// Argument corresponding to the NULL value.
size_t null_pos = temp_block.columns();
@ -68,15 +67,14 @@ public:
temp_block.insert(null_elem);
auto func_if = FunctionFactory::instance().get("if", context)->build(
{temp_block.getByPosition(res_pos), temp_block.getByPosition(null_pos), temp_block.getByPosition(arguments[0])});
func_if->execute(temp_block, {res_pos, arguments[0], null_pos}, result, input_rows_count);
{temp_block.getByPosition(equals_res_pos), temp_block.getByPosition(null_pos), temp_block.getByPosition(arguments[0])});
func_if->execute(temp_block, {equals_res_pos, null_pos, arguments[0]}, result, input_rows_count);
block.getByPosition(result).column = std::move(temp_block.getByPosition(result).column);
block.getByPosition(result).column = makeNullable(std::move(temp_block.getByPosition(result).column));
}
};
void registerFunctionNullIf(FunctionFactory & factory)
{
factory.registerFunction<FunctionNullIf>(FunctionFactory::CaseInsensitive);

View File

@ -38,6 +38,11 @@ public:
block.getByPosition(result).column
= DataTypeString().createColumnConst(input_rows_count, block.getByPosition(arguments[0]).column->getName());
}
ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block & block, const ColumnNumbers & arguments) const override
{
return DataTypeString().createColumnConst(1, block.getByPosition(arguments[0]).type->createColumn()->getName());
}
};

View File

@ -9,33 +9,16 @@ namespace DB
/** toTypeName(x) - get the type name
* Returns name of IDataType instance (name of data type).
*/
class FunctionToTypeName : public IFunction
class PreparedFunctionToTypeName : public PreparedFunctionImpl
{
public:
static constexpr auto name = "toTypeName";
static FunctionPtr create(const Context &)
{
return std::make_shared<FunctionToTypeName>();
}
String getName() const override
{
return name;
}
String getName() const override { return name; }
protected:
bool useDefaultImplementationForNulls() const override { return false; }
bool useDefaultImplementationForLowCardinalityColumns() const override { return false; }
size_t getNumberOfArguments() const override
{
return 1;
}
DataTypePtr getReturnTypeImpl(const DataTypes & /*arguments*/) const override
{
return std::make_shared<DataTypeString>();
}
/// Execute the function on the block.
void executeImpl(Block & block, const ColumnNumbers & arguments, size_t result, size_t input_rows_count) override
{
@ -45,9 +28,64 @@ public:
};
class BaseFunctionToTypeName : public IFunctionBase
{
public:
BaseFunctionToTypeName(DataTypes argument_types_, DataTypePtr return_type_)
: argument_types(std::move(argument_types_)), return_type(std::move(return_type_)) {}
static constexpr auto name = "toTypeName";
String getName() const override { return name; }
const DataTypes & getArgumentTypes() const override { return argument_types; }
const DataTypePtr & getReturnType() const override { return return_type; }
PreparedFunctionPtr prepare(const Block &, const ColumnNumbers &, size_t) const override
{
return std::make_shared<PreparedFunctionToTypeName>();
}
ColumnPtr getResultIfAlwaysReturnsConstantAndHasArguments(const Block &, const ColumnNumbers &) const override
{
return DataTypeString().createColumnConst(1, argument_types.at(0)->getName());
}
private:
DataTypes argument_types;
DataTypePtr return_type;
};
class FunctionToTypeNameBuilder : public FunctionBuilderImpl
{
public:
static constexpr auto name = "toTypeName";
String getName() const override { return name; }
static FunctionBuilderPtr create(const Context &) { return std::make_shared<FunctionToTypeNameBuilder>(); }
size_t getNumberOfArguments() const override { return 1; }
protected:
DataTypePtr getReturnTypeImpl(const DataTypes &) const override { return std::make_shared<DataTypeString>(); }
FunctionBasePtr buildImpl(const ColumnsWithTypeAndName & arguments, const DataTypePtr & return_type) const override
{
DataTypes types;
types.reserve(arguments.size());
for (auto & elem : arguments)
types.emplace_back(elem.type);
return std::make_shared<BaseFunctionToTypeName>(types, return_type);
}
bool useDefaultImplementationForNulls() const override { return false; }
bool useDefaultImplementationForLowCardinalityColumns() const override { return false; }
};
void registerFunctionToTypeName(FunctionFactory & factory)
{
factory.registerFunction<FunctionToTypeName>();
factory.registerFunction<FunctionToTypeNameBuilder>();
}
}

View File

@ -47,7 +47,7 @@ ReadBufferFromFile::ReadBufferFromFile(
if (o_direct)
{
if (fcntl(fd, F_NOCACHE, 1) == -1)
throwFromErrno("Cannot set F_NOCACHE on file " + file_name, file_name, ErrorCodes::CANNOT_OPEN_FILE);
throwFromErrnoWithPath("Cannot set F_NOCACHE on file " + file_name, file_name, ErrorCodes::CANNOT_OPEN_FILE);
}
#endif
}

View File

@ -51,7 +51,7 @@ WriteBufferFromFile::WriteBufferFromFile(
if (o_direct)
{
if (fcntl(fd, F_NOCACHE, 1) == -1)
throwFromErrno("Cannot set F_NOCACHE on file " + file_name, file_name, ErrorCodes::CANNOT_OPEN_FILE);
throwFromErrnoWithPath("Cannot set F_NOCACHE on file " + file_name, file_name, ErrorCodes::CANNOT_OPEN_FILE);
}
#endif
}

View File

@ -117,6 +117,9 @@ void WriteBufferValidUTF8::nextImpl()
memory[i] = p[i];
working_buffer = Buffer(&memory[cnt], memory.data() + memory.size());
/// Propagate next() to the output buffer
output_buffer.next();
}

View File

@ -68,7 +68,13 @@ BlockInputStreamPtr createLocalStream(const ASTPtr & query_ast, const Context &
* If you do not do this, different types (Const and non-Const) columns will be produced in different threads,
* And this is not allowed, since all code is based on the assumption that in the block stream all types are the same.
*/
return std::make_shared<MaterializingBlockInputStream>(stream);
/* Now we don't need to materialize constants, because RemoteBlockInputStream will ignore constant and take it from header.
* So, streams from different threads will always have the same header.
*/
/// return std::make_shared<MaterializingBlockInputStream>(stream);
return stream;
}
}

View File

@ -12,6 +12,7 @@
#include <Functions/IFunction.h>
#include <set>
#include <optional>
#include <DataTypes/DataTypeNullable.h>
namespace ProfileEvents
@ -159,20 +160,24 @@ ExpressionAction ExpressionAction::arrayJoin(const NameSet & array_joined_column
}
ExpressionAction ExpressionAction::ordinaryJoin(
const ASTTableJoin & join_params,
std::shared_ptr<const Join> join_,
const Names & join_key_names_left,
const Names & join_key_names_right,
const NamesAndTypesList & columns_added_by_join_)
{
ExpressionAction a;
a.type = JOIN;
a.join = std::move(join_);
a.join_kind = join_params.kind;
a.join_key_names_left = join_key_names_left;
a.join_key_names_right = join_key_names_right;
a.columns_added_by_join = columns_added_by_join_;
return a;
}
void ExpressionAction::prepare(Block & sample_block, const Settings & settings)
void ExpressionAction::prepare(Block & sample_block, const Settings & settings, NameSet & names_not_for_constant_folding)
{
// std::cerr << "preparing: " << toString() << std::endl;
@ -187,6 +192,7 @@ void ExpressionAction::prepare(Block & sample_block, const Settings & settings)
throw Exception("Column '" + result_name + "' already exists", ErrorCodes::DUPLICATE_COLUMN);
bool all_const = true;
bool all_suitable_for_constant_folding = true;
ColumnNumbers arguments(argument_names.size());
for (size_t i = 0; i < argument_names.size(); ++i)
@ -195,6 +201,9 @@ void ExpressionAction::prepare(Block & sample_block, const Settings & settings)
ColumnPtr col = sample_block.safeGetByPosition(arguments[i]).column;
if (!col || !isColumnConst(*col))
all_const = false;
if (names_not_for_constant_folding.count(argument_names[i]))
all_suitable_for_constant_folding = false;
}
size_t result_position = sample_block.columns();
@ -229,6 +238,22 @@ void ExpressionAction::prepare(Block & sample_block, const Settings & settings)
if (col.column->empty())
col.column = col.column->cloneResized(1);
if (!all_suitable_for_constant_folding)
names_not_for_constant_folding.insert(result_name);
}
}
/// Some functions like ignore() or getTypeName() always return constant result even if arguments are not constant.
/// We can't do constant folding, but can specify in sample block that function result is constant to avoid
/// unnecessary materialization.
auto & res = sample_block.getByPosition(result_position);
if (!res.column && function_base->isSuitableForConstantFolding())
{
if (auto col = function_base->getResultIfAlwaysReturnsConstantAndHasArguments(sample_block, arguments))
{
res.column = std::move(col);
names_not_for_constant_folding.insert(result_name);
}
}
@ -252,10 +277,50 @@ void ExpressionAction::prepare(Block & sample_block, const Settings & settings)
case JOIN:
{
/// TODO join_use_nulls setting
bool is_null_used_as_default = settings.join_use_nulls;
bool right_or_full_join = join_kind == ASTTableJoin::Kind::Right || join_kind == ASTTableJoin::Kind::Full;
bool left_or_full_join = join_kind == ASTTableJoin::Kind::Left || join_kind == ASTTableJoin::Kind::Full;
for (auto & col : sample_block)
{
/// Materialize column.
/// Column is not empty if it is constant, but after Join all constants will be materialized.
/// So, we need remove constants from header.
if (col.column)
col.column = nullptr;
bool make_nullable = is_null_used_as_default && right_or_full_join;
if (make_nullable && !col.type->isNullable())
col.type = std::make_shared<DataTypeNullable>(col.type);
}
for (const auto & col : columns_added_by_join)
sample_block.insert(ColumnWithTypeAndName(nullptr, col.type, col.name));
{
auto res_type = col.type;
bool make_nullable = is_null_used_as_default && left_or_full_join;
if (!make_nullable)
{
/// Keys from right table are usually not stored in Join, but copied from the left one.
/// So, if left key is nullable, let's make right key nullable too.
/// Note: for some join types it's not needed and, probably, may be removed.
/// Note: changing this code, take into account the implementation in Join.cpp.
auto it = std::find(join_key_names_right.begin(), join_key_names_right.end(), col.name);
if (it != join_key_names_right.end())
{
auto pos = it - join_key_names_right.begin();
const auto & left_key_name = join_key_names_left[pos];
make_nullable = sample_block.getByName(left_key_name).type->isNullable();
}
}
if (make_nullable && !res_type->isNullable())
res_type = std::make_shared<DataTypeNullable>(res_type);
sample_block.insert(ColumnWithTypeAndName(nullptr, res_type, col.name));
}
break;
}
@ -683,7 +748,7 @@ void ExpressionActions::addImpl(ExpressionAction action, Names & new_names)
for (const auto & name_with_alias : action.projection)
new_names.emplace_back(name_with_alias.second);
action.prepare(sample_block, settings);
action.prepare(sample_block, settings, names_not_for_constant_folding);
actions.push_back(action);
}
@ -915,7 +980,7 @@ void ExpressionActions::finalize(const Names & output_columns)
if (action.type == ExpressionAction::APPLY_FUNCTION && sample_block.has(out))
{
auto & result = sample_block.getByName(out);
if (result.column)
if (result.column && names_not_for_constant_folding.count(result.name) == 0)
{
action.type = ExpressionAction::ADD_COLUMN;
action.result_type = result.type;
@ -1262,6 +1327,7 @@ bool ExpressionAction::operator==(const ExpressionAction & other) const
&& array_join_is_left == other.array_join_is_left
&& join == other.join
&& join_key_names_left == other.join_key_names_left
&& join_key_names_right == other.join_key_names_right
&& columns_added_by_join == other.columns_added_by_join
&& projection == other.projection
&& is_function_compiled == other.is_function_compiled;

View File

@ -10,6 +10,7 @@
#include "config_core.h"
#include <unordered_map>
#include <unordered_set>
#include <Parsers/ASTTablesInSelectQuery.h>
namespace DB
@ -104,7 +105,9 @@ public:
/// For JOIN
std::shared_ptr<const Join> join;
ASTTableJoin::Kind join_kind;
Names join_key_names_left;
Names join_key_names_right;
NamesAndTypesList columns_added_by_join;
/// For PROJECT.
@ -121,7 +124,8 @@ public:
static ExpressionAction project(const Names & projected_columns_);
static ExpressionAction addAliases(const NamesWithAliases & aliased_columns_);
static ExpressionAction arrayJoin(const NameSet & array_joined_columns, bool array_join_is_left, const Context & context);
static ExpressionAction ordinaryJoin(std::shared_ptr<const Join> join_, const Names & join_key_names_left,
static ExpressionAction ordinaryJoin(const ASTTableJoin & join_params, std::shared_ptr<const Join> join_,
const Names & join_key_names_left, const Names & join_key_names_right,
const NamesAndTypesList & columns_added_by_join_);
/// Which columns necessary to perform this action.
@ -139,7 +143,7 @@ public:
private:
friend class ExpressionActions;
void prepare(Block & sample_block, const Settings & settings);
void prepare(Block & sample_block, const Settings & settings, NameSet & names_not_for_constant_folding);
void execute(Block & block, bool dry_run) const;
void executeOnTotals(Block & block) const;
};
@ -263,6 +267,8 @@ private:
Actions actions;
/// The example of result (output) block.
Block sample_block;
/// Columns which can't be used for constant folding.
NameSet names_not_for_constant_folding;
Settings settings;
#if USE_EMBEDDED_COMPILER

View File

@ -140,7 +140,7 @@ void ExpressionAnalyzer::analyzeAggregation()
for (const auto & key_ast : analyzedJoin().key_asts_left)
getRootActions(key_ast, true, temp_actions);
addJoinAction(temp_actions);
addJoinAction(table_join, temp_actions);
}
}
@ -424,9 +424,9 @@ static void appendRequiredColumns(
}
/// It's possible to set nullptr as join for only_types mode
void ExpressionAnalyzer::addJoinAction(ExpressionActionsPtr & actions, JoinPtr join) const
void ExpressionAnalyzer::addJoinAction(const ASTTableJoin & join_params, ExpressionActionsPtr & actions, JoinPtr join) const
{
actions->add(ExpressionAction::ordinaryJoin(join, analyzedJoin().key_names_left, columnsAddedByJoin()));
actions->add(ExpressionAction::ordinaryJoin(join_params, std::move(join), analyzedJoin().key_names_left, analyzedJoin().key_names_right, columnsAddedByJoin()));
}
bool SelectQueryExpressionAnalyzer::appendJoin(ExpressionActionsChain & chain, bool only_types)
@ -443,8 +443,10 @@ bool SelectQueryExpressionAnalyzer::appendJoin(ExpressionActionsChain & chain, b
initChain(chain, sourceColumns());
ExpressionActionsChain::Step & step = chain.steps.back();
auto & join_params = ast_join->table_join->as<ASTTableJoin &>();
getRootActions(left_keys_list, only_types, step.actions);
addJoinAction(step.actions, subquery_for_set.join);
addJoinAction(join_params, step.actions, subquery_for_set.join);
return true;
}

View File

@ -131,7 +131,7 @@ protected:
void addMultipleArrayJoinAction(ExpressionActionsPtr & actions, bool is_left) const;
void addJoinAction(ExpressionActionsPtr & actions, JoinPtr join = {}) const;
void addJoinAction(const ASTTableJoin & join_params, ExpressionActionsPtr & actions, JoinPtr join = {}) const;
void getRootActions(const ASTPtr & ast, bool no_subqueries, ExpressionActionsPtr & actions, bool only_consts = false);

View File

@ -400,6 +400,10 @@ public:
return count;
}
#if !__clang__
#pragma GCC diagnostic push
#pragma GCC diagnostic ignored "-Wunused-variable"
#endif
bool hasCurrentlyLoadedObjects() const
{
std::lock_guard lock{mutex};
@ -408,6 +412,9 @@ public:
return true;
return false;
}
#if !__clang__
#pragma GCC diagnostic pop
#endif
/// Starts loading of a specified object.
void load(const String & name)

View File

@ -8,7 +8,9 @@
#include <Storages/AlterCommands.h>
#include <Storages/MutationCommands.h>
#include <Storages/PartitionCommands.h>
#include <Storages/LiveViewCommands.h>
#include <Common/typeid_cast.h>
#include <Storages/StorageLiveView.h>
#include <algorithm>
@ -48,6 +50,7 @@ BlockIO InterpreterAlterQuery::execute()
AlterCommands alter_commands;
PartitionCommands partition_commands;
MutationCommands mutation_commands;
LiveViewCommands live_view_commands;
for (ASTAlterCommand * command_ast : alter.command_list->commands)
{
if (auto alter_command = AlterCommand::parse(command_ast))
@ -56,13 +59,16 @@ BlockIO InterpreterAlterQuery::execute()
partition_commands.emplace_back(std::move(*partition_command));
else if (auto mut_command = MutationCommand::parse(command_ast))
mutation_commands.emplace_back(std::move(*mut_command));
else if (auto live_view_command = LiveViewCommand::parse(command_ast))
live_view_commands.emplace_back(std::move(*live_view_command));
else
throw Exception("Wrong parameter type in ALTER query", ErrorCodes::LOGICAL_ERROR);
}
if (!mutation_commands.empty())
{
MutationsInterpreter(table, mutation_commands, context).validate();
auto table_lock_holder = table->lockStructureForShare(false /* because mutation is executed asyncronously */, context.getCurrentQueryId());
MutationsInterpreter(table, mutation_commands, context).validate(table_lock_holder);
table->mutate(mutation_commands, context);
}
@ -72,6 +78,21 @@ BlockIO InterpreterAlterQuery::execute()
table->alterPartition(query_ptr, partition_commands, context);
}
if (!live_view_commands.empty())
{
live_view_commands.validate(*table);
for (const LiveViewCommand & command : live_view_commands)
{
auto live_view = std::dynamic_pointer_cast<StorageLiveView>(table);
switch (command.type)
{
case LiveViewCommand::REFRESH:
live_view->refresh(context);
break;
}
}
}
if (!alter_commands.empty())
{
auto table_lock_holder = table->lockAlterIntention(context.getCurrentQueryId());

View File

@ -442,7 +442,7 @@ void InterpreterCreateQuery::setEngine(ASTCreateQuery & create) const
return;
}
if (create.temporary)
if (create.temporary && !create.is_live_view)
{
auto engine_ast = std::make_shared<ASTFunction>();
engine_ast->name = "Memory";
@ -465,6 +465,11 @@ void InterpreterCreateQuery::setEngine(ASTCreateQuery & create) const
"Cannot CREATE a table AS " + as_database_name + "." + as_table_name + ", it is a View",
ErrorCodes::INCORRECT_QUERY);
if (as_create.is_live_view)
throw Exception(
"Cannot CREATE a table AS " + as_database_name + "." + as_table_name + ", it is a Live View",
ErrorCodes::INCORRECT_QUERY);
create.set(create.storage, as_create.storage->ptr());
}
}
@ -482,7 +487,7 @@ BlockIO InterpreterCreateQuery::createTable(ASTCreateQuery & create)
}
/// Temporary tables are created out of databases.
if (create.temporary && !create.database.empty())
if (create.temporary && !create.database.empty() && !create.is_live_view)
throw Exception("Temporary tables cannot be inside a database. You should not specify a database for a temporary table.",
ErrorCodes::BAD_DATABASE_FOR_TEMPORARY_TABLE);
@ -505,7 +510,7 @@ BlockIO InterpreterCreateQuery::createTable(ASTCreateQuery & create)
if (create.to_database.empty())
create.to_database = current_database;
if (create.select && (create.is_view || create.is_materialized_view))
if (create.select && (create.is_view || create.is_materialized_view || create.is_live_view))
{
AddDefaultDatabaseVisitor visitor(current_database);
visitor.visit(*create.select);
@ -565,7 +570,7 @@ BlockIO InterpreterCreateQuery::createTable(ASTCreateQuery & create)
String data_path;
DatabasePtr database;
if (!create.temporary)
if (!create.temporary || create.is_live_view)
{
database = context.getDatabase(database_name);
data_path = database->getDataPath();
@ -611,7 +616,7 @@ BlockIO InterpreterCreateQuery::createTable(ASTCreateQuery & create)
false);
}
if (create.temporary)
if (create.temporary && !create.is_live_view)
context.getSessionContext().addExternalTable(table_name, res, query_ptr);
else
database->createTable(context, table_name, res, query_ptr);
@ -630,7 +635,7 @@ BlockIO InterpreterCreateQuery::createTable(ASTCreateQuery & create)
/// If the query is a CREATE SELECT, insert the data into the table.
if (create.select && !create.attach
&& !create.is_view && (!create.is_materialized_view || create.is_populate))
&& !create.is_view && !create.is_live_view && (!create.is_materialized_view || create.is_populate))
{
auto insert = std::make_shared<ASTInsertQuery>();

View File

@ -14,6 +14,7 @@
#include <Parsers/ASTUseQuery.h>
#include <Parsers/ASTExplainQuery.h>
#include <Parsers/TablePropertiesQueriesASTs.h>
#include <Parsers/ASTWatchQuery.h>
#include <Interpreters/InterpreterAlterQuery.h>
#include <Interpreters/InterpreterCheckQuery.h>
@ -35,6 +36,7 @@
#include <Interpreters/InterpreterShowTablesQuery.h>
#include <Interpreters/InterpreterSystemQuery.h>
#include <Interpreters/InterpreterUseQuery.h>
#include <Interpreters/InterpreterWatchQuery.h>
#include <Parsers/ASTSystemQuery.h>
@ -173,6 +175,10 @@ std::unique_ptr<IInterpreter> InterpreterFactory::get(ASTPtr & query, Context &
throwIfNoAccess(context);
return std::make_unique<InterpreterSystemQuery>(query, context);
}
else if (query->as<ASTWatchQuery>())
{
return std::make_unique<InterpreterWatchQuery>(query, context);
}
else
throw Exception("Unknown type of query: " + query->getID(), ErrorCodes::UNKNOWN_TYPE_OF_QUERY);
}

View File

@ -23,7 +23,6 @@ BlockIO InterpreterOptimizeQuery::execute()
return executeDDLQueryOnCluster(query_ptr, context, {ast.database});
StoragePtr table = context.getTable(ast.database, ast.table);
auto table_lock = table->lockStructureForShare(true, context.getCurrentQueryId());
table->optimize(query_ptr, ast.partition, ast.final, ast.deduplicate, context);
return {};
}

View File

@ -82,6 +82,8 @@
#include <Processors/Transforms/RollupTransform.h>
#include <Processors/Transforms/CubeTransform.h>
#include <Processors/LimitTransform.h>
#include <DataTypes/DataTypeAggregateFunction.h>
#include <DataStreams/materializeBlock.h>
namespace DB
@ -271,7 +273,7 @@ InterpreterSelectQuery::InterpreterSelectQuery(
String database_name;
String table_name;
getDatabaseAndTableNames(database_name, table_name);
getDatabaseAndTableNames(query, database_name, table_name, context);
if (auto view_source = context.getViewSource())
{
@ -345,17 +347,20 @@ InterpreterSelectQuery::InterpreterSelectQuery(
source_header = storage->getSampleBlockForColumns(required_columns);
/// Calculate structure of the result.
result_header = getSampleBlockImpl();
for (auto & col : result_header)
{
Pipeline pipeline;
executeImpl(pipeline, nullptr, true);
result_header = pipeline.firstStream()->getHeader();
if (!col.column)
col.column = col.type->createColumn();
else if (isColumnConst(*col.column) && !col.column->empty())
col.column = col.column->cloneEmpty();
}
}
void InterpreterSelectQuery::getDatabaseAndTableNames(String & database_name, String & table_name)
void InterpreterSelectQuery::getDatabaseAndTableNames(const ASTSelectQuery & query, String & database_name, String & table_name, const Context & context)
{
if (auto db_and_table = getDatabaseAndTable(getSelectQuery(), 0))
if (auto db_and_table = getDatabaseAndTable(query, 0))
{
table_name = db_and_table->table;
database_name = db_and_table->database;
@ -381,8 +386,8 @@ Block InterpreterSelectQuery::getSampleBlock()
BlockIO InterpreterSelectQuery::execute()
{
Pipeline pipeline;
executeImpl(pipeline, input, options.only_analyze);
executeUnion(pipeline);
executeImpl(pipeline, input);
executeUnion(pipeline, getSampleBlock());
BlockIO res;
res.in = pipeline.firstStream();
@ -392,28 +397,104 @@ BlockIO InterpreterSelectQuery::execute()
BlockInputStreams InterpreterSelectQuery::executeWithMultipleStreams()
{
Pipeline pipeline;
executeImpl(pipeline, input, options.only_analyze);
executeImpl(pipeline, input);
unifyStreams(pipeline, getSampleBlock());
return pipeline.streams;
}
QueryPipeline InterpreterSelectQuery::executeWithProcessors()
{
QueryPipeline query_pipeline;
executeImpl(query_pipeline, input, options.only_analyze);
executeImpl(query_pipeline, input);
return query_pipeline;
}
Block InterpreterSelectQuery::getSampleBlockImpl()
{
FilterInfoPtr filter_info;
/// Need to create sets before analyzeExpressions(). Otherwise some sets for index won't be created.
query_analyzer->makeSetsForIndex(getSelectQuery().where());
query_analyzer->makeSetsForIndex(getSelectQuery().prewhere());
auto analysis_result = analyzeExpressions(
getSelectQuery(),
*query_analyzer,
QueryProcessingStage::Enum::FetchColumns,
options.to_stage,
context,
storage,
true,
filter_info);
if (options.to_stage == QueryProcessingStage::Enum::FetchColumns)
{
auto header = source_header;
if (analysis_result.prewhere_info)
{
analysis_result.prewhere_info->prewhere_actions->execute(header);
header = materializeBlock(header);
if (analysis_result.prewhere_info->remove_prewhere_column)
header.erase(analysis_result.prewhere_info->prewhere_column_name);
}
return header;
}
if (options.to_stage == QueryProcessingStage::Enum::WithMergeableState)
{
if (!analysis_result.need_aggregate)
return analysis_result.before_order_and_select->getSampleBlock();
auto header = analysis_result.before_aggregation->getSampleBlock();
Names key_names;
AggregateDescriptions aggregates;
query_analyzer->getAggregateInfo(key_names, aggregates);
Block res;
for (auto & key : key_names)
res.insert({nullptr, header.getByName(key).type, key});
for (auto & aggregate : aggregates)
{
size_t arguments_size = aggregate.argument_names.size();
DataTypes argument_types(arguments_size);
for (size_t j = 0; j < arguments_size; ++j)
argument_types[j] = header.getByName(aggregate.argument_names[j]).type;
DataTypePtr type = std::make_shared<DataTypeAggregateFunction>(aggregate.function, argument_types, aggregate.parameters);
res.insert({nullptr, type, aggregate.column_name});
}
return res;
}
return analysis_result.final_projection->getSampleBlock();
}
InterpreterSelectQuery::AnalysisResult
InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage, bool dry_run, const FilterInfoPtr & filter_info)
InterpreterSelectQuery::analyzeExpressions(
const ASTSelectQuery & query,
SelectQueryExpressionAnalyzer & query_analyzer,
QueryProcessingStage::Enum from_stage,
QueryProcessingStage::Enum to_stage,
const Context & context,
const StoragePtr & storage,
bool only_types,
const FilterInfoPtr & filter_info)
{
AnalysisResult res;
/// Do I need to perform the first part of the pipeline - running on remote servers during distributed processing.
res.first_stage = from_stage < QueryProcessingStage::WithMergeableState
&& options.to_stage >= QueryProcessingStage::WithMergeableState;
&& to_stage >= QueryProcessingStage::WithMergeableState;
/// Do I need to execute the second part of the pipeline - running on the initiating server during distributed processing.
res.second_stage = from_stage <= QueryProcessingStage::WithMergeableState
&& options.to_stage > QueryProcessingStage::WithMergeableState;
&& to_stage > QueryProcessingStage::WithMergeableState;
/** First we compose a chain of actions and remember the necessary steps from it.
* Regardless of from_stage and to_stage, we will compose a complete sequence of actions to perform optimization and
@ -468,8 +549,6 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
{
ExpressionActionsChain chain(context);
auto & query = getSelectQuery();
Names additional_required_columns_after_prewhere;
if (storage && query.sample_size())
@ -486,14 +565,14 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
columns_for_final.begin(), columns_for_final.end());
}
if (storage && context.hasUserProperty(storage->getDatabaseName(), storage->getTableName(), "filter"))
if (storage && filter_info)
{
has_filter = true;
/// XXX: aggregated copy-paste from ExpressionAnalyzer::appendSmth()
if (chain.steps.empty())
{
chain.steps.emplace_back(std::make_shared<ExpressionActions>(source_columns, context));
chain.steps.emplace_back(std::make_shared<ExpressionActions>(NamesAndTypesList(), context));
}
ExpressionActionsChain::Step & step = chain.steps.back();
@ -506,7 +585,7 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
chain.addStep();
}
if (query_analyzer->appendPrewhere(chain, !res.first_stage, additional_required_columns_after_prewhere))
if (query_analyzer.appendPrewhere(chain, !res.first_stage, additional_required_columns_after_prewhere))
{
has_prewhere = true;
@ -516,11 +595,11 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
chain.addStep();
}
res.need_aggregate = query_analyzer->hasAggregation();
res.need_aggregate = query_analyzer.hasAggregation();
query_analyzer->appendArrayJoin(chain, dry_run || !res.first_stage);
query_analyzer.appendArrayJoin(chain, only_types || !res.first_stage);
if (query_analyzer->appendJoin(chain, dry_run || !res.first_stage))
if (query_analyzer.appendJoin(chain, only_types || !res.first_stage))
{
res.before_join = chain.getLastActions();
if (!res.hasJoin())
@ -528,7 +607,7 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
chain.addStep();
}
if (query_analyzer->appendWhere(chain, dry_run || !res.first_stage))
if (query_analyzer.appendWhere(chain, only_types || !res.first_stage))
{
where_step_num = chain.steps.size() - 1;
has_where = res.has_where = true;
@ -538,13 +617,13 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
if (res.need_aggregate)
{
query_analyzer->appendGroupBy(chain, dry_run || !res.first_stage);
query_analyzer->appendAggregateFunctionsArguments(chain, dry_run || !res.first_stage);
query_analyzer.appendGroupBy(chain, only_types || !res.first_stage);
query_analyzer.appendAggregateFunctionsArguments(chain, only_types || !res.first_stage);
res.before_aggregation = chain.getLastActions();
finalizeChain(chain);
if (query_analyzer->appendHaving(chain, dry_run || !res.second_stage))
if (query_analyzer.appendHaving(chain, only_types || !res.second_stage))
{
res.has_having = true;
res.before_having = chain.getLastActions();
@ -553,20 +632,20 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
}
/// If there is aggregation, we execute expressions in SELECT and ORDER BY on the initiating server, otherwise on the source servers.
query_analyzer->appendSelect(chain, dry_run || (res.need_aggregate ? !res.second_stage : !res.first_stage));
query_analyzer.appendSelect(chain, only_types || (res.need_aggregate ? !res.second_stage : !res.first_stage));
res.selected_columns = chain.getLastStep().required_output;
res.has_order_by = query_analyzer->appendOrderBy(chain, dry_run || (res.need_aggregate ? !res.second_stage : !res.first_stage));
res.has_order_by = query_analyzer.appendOrderBy(chain, only_types || (res.need_aggregate ? !res.second_stage : !res.first_stage));
res.before_order_and_select = chain.getLastActions();
chain.addStep();
if (query_analyzer->appendLimitBy(chain, dry_run || !res.second_stage))
if (query_analyzer.appendLimitBy(chain, only_types || !res.second_stage))
{
res.has_limit_by = true;
res.before_limit_by = chain.getLastActions();
chain.addStep();
}
query_analyzer->appendProjectResult(chain);
query_analyzer.appendProjectResult(chain);
res.final_projection = chain.getLastActions();
finalizeChain(chain);
@ -580,7 +659,7 @@ InterpreterSelectQuery::analyzeExpressions(QueryProcessingStage::Enum from_stage
if (res.has_having)
res.before_having->prependProjectInput();
res.subqueries_for_sets = query_analyzer->getSubqueriesForSets();
res.subqueries_for_sets = query_analyzer.getSubqueriesForSets();
/// Check that PREWHERE doesn't contain unusual actions. Unusual actions are that can change number of rows.
if (res.prewhere_info)
@ -747,7 +826,7 @@ static SortingInfoPtr optimizeReadInOrder(const MergeTreeData & merge_tree, cons
template <typename TPipeline>
void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputStreamPtr & prepared_input, bool dry_run)
void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputStreamPtr & prepared_input)
{
/** Streams of data. When the query is executed in parallel, we have several data streams.
* If there is no GROUP BY, then perform all operations before ORDER BY and LIMIT in parallel, then
@ -771,7 +850,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS
/// Turn off, if the table filter is applied.
if (storage && !context.hasUserProperty(storage->getDatabaseName(), storage->getTableName(), "filter"))
{
if (!dry_run)
if (!options.only_analyze)
from_stage = storage->getQueryProcessingStage(context);
query_analyzer->makeSetsForIndex(query.where());
@ -811,14 +890,22 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS
sorting_info = optimizeReadInOrder(*merge_tree_data, query, context, syntax_analyzer_result);
}
if (dry_run)
if (options.only_analyze)
{
if constexpr (pipeline_with_processors)
pipeline.init({std::make_shared<NullSource>(source_header)});
else
pipeline.streams.emplace_back(std::make_shared<NullBlockInputStream>(source_header));
expressions = analyzeExpressions(QueryProcessingStage::FetchColumns, true, filter_info);
expressions = analyzeExpressions(
getSelectQuery(),
*query_analyzer,
QueryProcessingStage::FetchColumns,
options.to_stage,
context,
storage,
true,
filter_info);
if (storage && expressions.filter_info && expressions.prewhere_info)
throw Exception("PREWHERE is not supported if the table is filtered by row-level security expression", ErrorCodes::ILLEGAL_PREWHERE);
@ -850,7 +937,15 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS
pipeline.streams.push_back(prepared_input);
}
expressions = analyzeExpressions(from_stage, false, filter_info);
expressions = analyzeExpressions(
getSelectQuery(),
*query_analyzer,
from_stage,
options.to_stage,
context,
storage,
false,
filter_info);
if (from_stage == QueryProcessingStage::WithMergeableState &&
options.to_stage == QueryProcessingStage::WithMergeableState)
@ -1097,7 +1192,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS
if constexpr (pipeline_with_processors)
pipeline.resize(1);
else
executeUnion(pipeline);
executeUnion(pipeline, {});
}
/** If there was more than one stream,
@ -1696,7 +1791,7 @@ void InterpreterSelectQuery::executeMergeAggregated(Pipeline & pipeline, bool ov
if (!settings.distributed_aggregation_memory_efficient)
{
/// We union several sources into one, parallelizing the work.
executeUnion(pipeline);
executeUnion(pipeline, {});
/// Now merge the aggregated blocks
pipeline.firstStream() = std::make_shared<MergingAggregatedBlockInputStream>(pipeline.firstStream(), params, final, settings.max_threads);
@ -1798,7 +1893,7 @@ void InterpreterSelectQuery::executeHaving(QueryPipeline & pipeline, const Expre
void InterpreterSelectQuery::executeTotalsAndHaving(Pipeline & pipeline, bool has_having, const ExpressionActionsPtr & expression, bool overflow_row, bool final)
{
executeUnion(pipeline);
executeUnion(pipeline, {});
const Settings & settings = context.getSettingsRef();
@ -1827,7 +1922,7 @@ void InterpreterSelectQuery::executeTotalsAndHaving(QueryPipeline & pipeline, bo
void InterpreterSelectQuery::executeRollupOrCube(Pipeline & pipeline, Modificator modificator)
{
executeUnion(pipeline);
executeUnion(pipeline, {});
Names key_names;
AggregateDescriptions aggregates;
@ -1972,7 +2067,7 @@ void InterpreterSelectQuery::executeOrder(Pipeline & pipeline, SortingInfoPtr so
});
/// If there are several streams, we merge them into one
executeUnion(pipeline);
executeUnion(pipeline, {});
/// Merge the sorted blocks.
pipeline.firstStream() = std::make_shared<MergeSortingBlockInputStream>(
@ -2032,7 +2127,7 @@ void InterpreterSelectQuery::executeMergeSorted(Pipeline & pipeline)
/// If there are several streams, then we merge them into one
if (pipeline.hasMoreThanOneStream())
{
unifyStreams(pipeline);
unifyStreams(pipeline, pipeline.firstStream()->getHeader());
/** MergingSortedBlockInputStream reads the sources sequentially.
* To make the data on the remote servers prepared in parallel, we wrap it in AsynchronousBlockInputStream.
@ -2136,12 +2231,15 @@ void InterpreterSelectQuery::executeDistinct(QueryPipeline & pipeline, bool befo
}
void InterpreterSelectQuery::executeUnion(Pipeline & pipeline)
void InterpreterSelectQuery::executeUnion(Pipeline & pipeline, Block header)
{
/// If there are still several streams, then we combine them into one
if (pipeline.hasMoreThanOneStream())
{
unifyStreams(pipeline);
if (!header)
header = pipeline.firstStream()->getHeader();
unifyStreams(pipeline, std::move(header));
pipeline.firstStream() = std::make_shared<UnionBlockInputStream>(pipeline.streams, pipeline.stream_with_non_joined_data, max_streams);
pipeline.stream_with_non_joined_data = nullptr;
@ -2351,7 +2449,7 @@ void InterpreterSelectQuery::executeExtremes(QueryPipeline & pipeline)
void InterpreterSelectQuery::executeSubqueriesInSetsAndJoins(Pipeline & pipeline, SubqueriesForSets & subqueries_for_sets)
{
executeUnion(pipeline);
executeUnion(pipeline, {});
pipeline.firstStream() = std::make_shared<CreatingSetsBlockInputStream>(
pipeline.firstStream(), subqueries_for_sets, context);
}
@ -2369,20 +2467,22 @@ void InterpreterSelectQuery::executeSubqueriesInSetsAndJoins(QueryPipeline & pip
}
void InterpreterSelectQuery::unifyStreams(Pipeline & pipeline)
void InterpreterSelectQuery::unifyStreams(Pipeline & pipeline, Block header)
{
if (pipeline.hasMoreThanOneStream())
/// Unify streams in case they have different headers.
/// TODO: remove previos addition of _dummy column.
if (header.columns() > 1 && header.has("_dummy"))
header.erase("_dummy");
for (size_t i = 0; i < pipeline.streams.size(); ++i)
{
/// Unify streams in case they have different headers.
auto first_header = pipeline.streams.at(0)->getHeader();
for (size_t i = 1; i < pipeline.streams.size(); ++i)
{
auto & stream = pipeline.streams[i];
auto header = stream->getHeader();
auto mode = ConvertingBlockInputStream::MatchColumnsMode::Name;
if (!blocksHaveEqualStructure(first_header, header))
stream = std::make_shared<ConvertingBlockInputStream>(context, stream, first_header, mode);
}
auto & stream = pipeline.streams[i];
auto stream_header = stream->getHeader();
auto mode = ConvertingBlockInputStream::MatchColumnsMode::Name;
if (!blocksHaveEqualStructure(header, stream_header))
stream = std::make_shared<ConvertingBlockInputStream>(context, stream, header, mode);
}
}

View File

@ -90,6 +90,7 @@ private:
ASTSelectQuery & getSelectQuery() { return query_ptr->as<ASTSelectQuery &>(); }
Block getSampleBlockImpl();
struct Pipeline
{
@ -135,7 +136,7 @@ private:
};
template <typename TPipeline>
void executeImpl(TPipeline & pipeline, const BlockInputStreamPtr & prepared_input, bool dry_run);
void executeImpl(TPipeline & pipeline, const BlockInputStreamPtr & prepared_input);
struct AnalysisResult
{
@ -172,12 +173,19 @@ private:
FilterInfoPtr filter_info;
};
AnalysisResult analyzeExpressions(QueryProcessingStage::Enum from_stage, bool dry_run, const FilterInfoPtr & filter_info);
static AnalysisResult analyzeExpressions(
const ASTSelectQuery & query,
SelectQueryExpressionAnalyzer & query_analyzer,
QueryProcessingStage::Enum from_stage,
QueryProcessingStage::Enum to_stage,
const Context & context,
const StoragePtr & storage,
bool only_types,
const FilterInfoPtr & filter_info);
/** From which table to read. With JOIN, the "left" table is returned.
*/
void getDatabaseAndTableNames(String & database_name, String & table_name);
static void getDatabaseAndTableNames(const ASTSelectQuery & query, String & database_name, String & table_name, const Context & context);
/// Different stages of query execution.
@ -198,7 +206,7 @@ private:
void executeOrder(Pipeline & pipeline, SortingInfoPtr sorting_info);
void executeMergeSorted(Pipeline & pipeline);
void executePreLimit(Pipeline & pipeline);
void executeUnion(Pipeline & pipeline);
void executeUnion(Pipeline & pipeline, Block header); /// If header is not empty, convert streams structure to it.
void executeLimitBy(Pipeline & pipeline);
void executeLimit(Pipeline & pipeline);
void executeProjection(Pipeline & pipeline, const ExpressionActionsPtr & expression);
@ -222,8 +230,8 @@ private:
void executeExtremes(QueryPipeline & pipeline);
void executeSubqueriesInSetsAndJoins(QueryPipeline & pipeline, std::unordered_map<String, SubqueryForSet> & subqueries_for_sets);
/// If pipeline has several streams with different headers, add ConvertingBlockInputStream to first header.
void unifyStreams(Pipeline & pipeline);
/// Add ConvertingBlockInputStream to specified header.
void unifyStreams(Pipeline & pipeline, Block header);
enum class Modificator
{
@ -246,7 +254,6 @@ private:
const SelectQueryOptions options;
ASTPtr query_ptr;
Context context;
NamesAndTypesList source_columns;
SyntaxAnalyzerResultPtr syntax_analyzer_result;
std::unique_ptr<SelectQueryExpressionAnalyzer> query_analyzer;
SelectQueryInfo query_info;

View File

@ -0,0 +1,108 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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 <Core/Settings.h>
#include <Common/typeid_cast.h>
#include <Parsers/ASTWatchQuery.h>
#include <Interpreters/InterpreterWatchQuery.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataStreams/OneBlockInputStream.h>
namespace DB
{
namespace ErrorCodes
{
extern const int UNKNOWN_STORAGE;
extern const int UNKNOWN_TABLE;
extern const int TOO_MANY_COLUMNS;
}
BlockInputStreamPtr InterpreterWatchQuery::executeImpl()
{
return std::make_shared<OneBlockInputStream>(Block());
}
BlockIO InterpreterWatchQuery::execute()
{
BlockIO res;
const ASTWatchQuery & query = typeid_cast<const ASTWatchQuery &>(*query_ptr);
String database;
String table;
/// Get database
if (!query.database.empty())
database = query.database;
else
database = context.getCurrentDatabase();
/// Get table
table = query.table;
/// Get storage
storage = context.tryGetTable(database, table);
if (!storage)
throw Exception("Table " + backQuoteIfNeed(database) + "." +
backQuoteIfNeed(table) + " doesn't exist.",
ErrorCodes::UNKNOWN_TABLE);
/// List of columns to read to execute the query.
Names required_columns = storage->getColumns().getNamesOfPhysical();
/// Get context settings for this query
const Settings & settings = context.getSettingsRef();
/// Limitation on the number of columns to read.
if (settings.max_columns_to_read && required_columns.size() > settings.max_columns_to_read)
throw Exception("Limit for number of columns to read exceeded. "
"Requested: " + std::to_string(required_columns.size())
+ ", maximum: " + settings.max_columns_to_read.toString(),
ErrorCodes::TOO_MANY_COLUMNS);
size_t max_block_size = settings.max_block_size;
size_t max_streams = 1;
/// Define query info
SelectQueryInfo query_info;
query_info.query = query_ptr;
/// From stage
QueryProcessingStage::Enum from_stage = QueryProcessingStage::FetchColumns;
QueryProcessingStage::Enum to_stage = QueryProcessingStage::Complete;
/// Watch storage
streams = storage->watch(required_columns, query_info, context, from_stage, max_block_size, max_streams);
/// Constraints on the result, the quota on the result, and also callback for progress.
if (IBlockInputStream * stream = dynamic_cast<IBlockInputStream *>(streams[0].get()))
{
/// Constraints apply only to the final result.
if (to_stage == QueryProcessingStage::Complete)
{
IBlockInputStream::LocalLimits limits;
limits.mode = IBlockInputStream::LIMITS_CURRENT;
limits.size_limits.max_rows = settings.max_result_rows;
limits.size_limits.max_bytes = settings.max_result_bytes;
limits.size_limits.overflow_mode = settings.result_overflow_mode;
stream->setLimits(limits);
stream->setQuota(context.getQuota());
}
}
res.in = streams[0];
return res;
}
}

View File

@ -0,0 +1,50 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <Core/QueryProcessingStage.h>
#include <DataStreams/BlockIO.h>
#include <DataStreams/IBlockInputStream.h>
#include <Parsers/IAST_fwd.h>
#include <Interpreters/IInterpreter.h>
#include <Storages/SelectQueryInfo.h>
#include <Storages/IStorage.h>
#include <Interpreters/Context.h>
namespace DB
{
class IAST;
using ASTPtr = std::shared_ptr<IAST>;
using StoragePtr = std::shared_ptr<IStorage>;
class InterpreterWatchQuery : public IInterpreter
{
public:
InterpreterWatchQuery(const ASTPtr & query_ptr_, Context & context_)
: query_ptr(query_ptr_), context(context_) {}
BlockIO execute() override;
private:
ASTPtr query_ptr;
Context & context;
BlockInputStreamPtr executeImpl();
/// Table from where to read data, if not subquery.
StoragePtr storage;
/// Streams of read data
BlockInputStreams streams;
};
}

View File

@ -126,6 +126,8 @@ public:
bool empty() { return type == Type::EMPTY; }
bool isNullUsedAsDefault() const { return use_nulls; }
/** Set information about structure of right hand of JOIN (joined data).
* You must call this method before subsequent calls to insertFromBlock.
*/
@ -168,6 +170,7 @@ public:
size_t getTotalByteCount() const;
ASTTableJoin::Kind getKind() const { return kind; }
ASTTableJoin::Strictness getStrictness() const { return strictness; }
AsofRowRefs::Type getAsofType() const { return *asof_type; }
bool anyTakeLastRow() const { return any_take_last_row; }

View File

@ -458,15 +458,16 @@ BlockInputStreamPtr MutationsInterpreter::addStreamsForLaterStages(const std::ve
return in;
}
void MutationsInterpreter::validate()
void MutationsInterpreter::validate(TableStructureReadLockHolder &)
{
prepare(/* dry_run = */ true);
Block first_stage_header = interpreter_select->getSampleBlock();
/// Do not use getSampleBlock in order to check the whole pipeline.
Block first_stage_header = interpreter_select->execute().in->getHeader();
BlockInputStreamPtr in = std::make_shared<NullBlockInputStream>(first_stage_header);
addStreamsForLaterStages(stages, in)->getHeader();
}
BlockInputStreamPtr MutationsInterpreter::execute()
BlockInputStreamPtr MutationsInterpreter::execute(TableStructureReadLockHolder &)
{
prepare(/* dry_run = */ false);
BlockInputStreamPtr in = interpreter_select->execute().in;

View File

@ -25,13 +25,13 @@ public:
{
}
void validate();
void validate(TableStructureReadLockHolder & table_lock_holder);
/// Return false if the data isn't going to be changed by mutations.
bool isStorageTouchedByMutations() const;
/// The resulting stream will return blocks containing only changed columns and columns, that we need to recalculate indices.
BlockInputStreamPtr execute();
BlockInputStreamPtr execute(TableStructureReadLockHolder & table_lock_holder);
/// Only changed columns.
const Block & getUpdatedHeader() const;
@ -44,7 +44,6 @@ private:
std::unique_ptr<InterpreterSelectQuery> prepareInterpreterSelect(std::vector<Stage> & prepared_stages, bool dry_run);
BlockInputStreamPtr addStreamsForLaterStages(const std::vector<Stage> & prepared_stages, BlockInputStreamPtr in) const;
private:
StoragePtr storage;
std::vector<MutationCommand> commands;
const Context & context;

View File

@ -102,8 +102,24 @@ void QueryNormalizer::visit(ASTIdentifier & node, ASTPtr & ast, Data & data)
/// If it is an alias, but not a parent alias (for constructs like "SELECT column + 1 AS column").
auto it_alias = data.aliases.find(node.name);
if (IdentifierSemantic::canBeAlias(node) && it_alias != data.aliases.end() && current_alias != node.name)
if (it_alias != data.aliases.end() && current_alias != node.name)
{
if (!IdentifierSemantic::canBeAlias(node))
{
/// This means that column had qualified name, which was translated (so, canBeAlias() returns false).
/// But there is an alias with the same name. So, let's use original name for that column.
/// If alias wasn't set, use original column name as alias.
/// That helps to avoid result set with columns which have same names but different values.
if (node.alias.empty())
{
node.name.swap(node.alias);
node.restoreCompoundName();
node.name.swap(node.alias);
}
return;
}
auto & alias_node = it_alias->second;
/// Let's replace it with the corresponding tree node.

41
dbms/src/NOTICE Normal file
View File

@ -0,0 +1,41 @@
--
The following notice shall be applied to the files listed below.
Some modifications Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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.
Common/ErrorCodes.cpp
Common/UInt128.h
Core/Block.h
Core/Defines.h
Core/Settings.h
DataStreams/PushingToViewsBlockOutputStream.cpp
DataStreams/PushingToViewsBlockOutputStream.h
DataStreams/copyData.cpp
Databases/DatabasesCommon.cpp
IO/WriteBufferValidUTF8.cpp
Interpreters/InterpreterAlterQuery.cpp
Interpreters/InterpreterCreateQuery.cpp
Interpreters/InterpreterFactory.cpp
Parsers/ASTAlterQuery.cpp
Parsers/ASTAlterQuery.h
Parsers/ASTCreateQuery.cpp
Parsers/ASTCreateQuery.h
Parsers/ParserAlterQuery.cpp
Parsers/ParserAlterQuery.h
Parsers/ParserCreateQuery.cpp
Parsers/ParserCreateQuery.h
Parsers/ParserQueryWithOutput.cpp
Storages/IStorage.h
Storages/StorageFactory.cpp
Storages/registerStorages.cpp
--

View File

@ -45,6 +45,11 @@ ASTPtr ASTAlterCommand::clone() const
res->ttl = ttl->clone();
res->children.push_back(res->ttl);
}
if (values)
{
res->values = values->clone();
res->children.push_back(res->values);
}
return res;
}
@ -200,6 +205,46 @@ void ASTAlterCommand::formatImpl(
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "MODIFY TTL " << (settings.hilite ? hilite_none : "");
ttl->formatImpl(settings, state, frame);
}
else if (type == ASTAlterCommand::LIVE_VIEW_REFRESH)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "REFRESH " << (settings.hilite ? hilite_none : "");
}
else if (type == ASTAlterCommand::LIVE_CHANNEL_ADD)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "ADD " << (settings.hilite ? hilite_none : "");
values->formatImpl(settings, state, frame);
}
else if (type == ASTAlterCommand::LIVE_CHANNEL_DROP)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "DROP " << (settings.hilite ? hilite_none : "");
values->formatImpl(settings, state, frame);
}
else if (type == ASTAlterCommand::LIVE_CHANNEL_MODIFY)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "MODIFY " << (settings.hilite ? hilite_none : "");
values->formatImpl(settings, state, frame);
}
else if (type == ASTAlterCommand::LIVE_CHANNEL_SUSPEND)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "SUSPEND " << (settings.hilite ? hilite_none : "");
values->formatImpl(settings, state, frame);
}
else if (type == ASTAlterCommand::LIVE_CHANNEL_RESUME)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "RESUME " << (settings.hilite ? hilite_none : "");
values->formatImpl(settings, state, frame);
}
else if (type == ASTAlterCommand::LIVE_CHANNEL_REFRESH)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "REFRESH " << (settings.hilite ? hilite_none : "");
values->formatImpl(settings, state, frame);
}
else
throw Exception("Unexpected type of ALTER", ErrorCodes::UNEXPECTED_AST_STRUCTURE);
}
@ -252,7 +297,12 @@ void ASTAlterQuery::formatQueryImpl(const FormatSettings & settings, FormatState
std::string indent_str = settings.one_line ? "" : std::string(4u * frame.indent, ' ');
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "ALTER TABLE " << (settings.hilite ? hilite_none : "");
if (is_live_view)
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "ALTER LIVE VIEW " << (settings.hilite ? hilite_none : "");
else if (is_live_channel)
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "ALTER LIVE CHANNEL " << (settings.hilite ? hilite_none : "");
else
settings.ostr << (settings.hilite ? hilite_keyword : "") << indent_str << "ALTER TABLE " << (settings.hilite ? hilite_none : "");
if (!table.empty())
{

View File

@ -15,6 +15,15 @@ namespace DB
* MODIFY COLUMN col_name type,
* DROP PARTITION partition,
* COMMENT_COLUMN col_name 'comment',
* ALTER LIVE VIEW [db.]name_type
* REFRESH
* ALTER CHANNEL [db.]name_type
* ADD live_view,...
* DROP live_view,...
* SUSPEND live_view,...
* RESUME live_view,...
* REFRESH live_view,...
* MODIFY live_view,...
*/
class ASTAlterCommand : public IAST
@ -44,6 +53,15 @@ public:
UPDATE,
NO_TYPE,
LIVE_VIEW_REFRESH,
LIVE_CHANNEL_ADD,
LIVE_CHANNEL_DROP,
LIVE_CHANNEL_SUSPEND,
LIVE_CHANNEL_RESUME,
LIVE_CHANNEL_REFRESH,
LIVE_CHANNEL_MODIFY
};
Type type = NO_TYPE;
@ -91,6 +109,10 @@ public:
/// For MODIFY TTL query
ASTPtr ttl;
/** In ALTER CHANNEL, ADD, DROP, SUSPEND, RESUME, REFRESH, MODIFY queries, the list of live views is stored here
*/
ASTPtr values;
bool detach = false; /// true for DETACH PARTITION
bool part = false; /// true for ATTACH PART
@ -147,6 +169,9 @@ protected:
class ASTAlterQuery : public ASTQueryWithTableAndOutput, public ASTQueryWithOnCluster
{
public:
bool is_live_view{false}; /// true for ALTER LIVE VIEW
bool is_live_channel{false}; /// true for ALTER LIVE CHANNEL
ASTAlterCommandList * command_list = nullptr;
String getID(char) const override;

View File

@ -173,6 +173,8 @@ ASTPtr ASTCreateQuery::clone() const
res->set(res->storage, storage->clone());
if (select)
res->set(res->select, select->clone());
if (tables)
res->set(res->tables, tables->clone());
cloneOutputOptions(*res);
@ -204,6 +206,11 @@ void ASTCreateQuery::formatQueryImpl(const FormatSettings & settings, FormatStat
what = "VIEW";
if (is_materialized_view)
what = "MATERIALIZED VIEW";
if (is_live_view)
what = "LIVE VIEW";
if (is_live_channel)
what = "LIVE CHANNEL";
settings.ostr
<< (settings.hilite ? hilite_keyword : "")
@ -257,6 +264,12 @@ void ASTCreateQuery::formatQueryImpl(const FormatSettings & settings, FormatStat
settings.ostr << (settings.hilite ? hilite_keyword : "") << " AS" << settings.nl_or_ws << (settings.hilite ? hilite_none : "");
select->formatImpl(settings, state, frame);
}
if (tables)
{
settings.ostr << (settings.hilite ? hilite_keyword : "") << " WITH " << (settings.hilite ? hilite_none : "");
tables->formatImpl(settings, state, frame);
}
}
}

View File

@ -55,9 +55,12 @@ public:
bool if_not_exists{false};
bool is_view{false};
bool is_materialized_view{false};
bool is_live_view{false};
bool is_live_channel{false};
bool is_populate{false};
bool replace_view{false}; /// CREATE OR REPLACE VIEW
ASTColumns * columns_list = nullptr;
ASTExpressionList *tables = nullptr;
String to_database; /// For CREATE MATERIALIZED VIEW mv TO table.
String to_table;
ASTStorage * storage = nullptr;

View File

@ -0,0 +1,59 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <Parsers/ASTQueryWithTableAndOutput.h>
namespace DB
{
class ASTWatchQuery : public ASTQueryWithTableAndOutput
{
public:
ASTPtr limit_length;
bool is_watch_events;
ASTWatchQuery() = default;
String getID(char) const override { return "WatchQuery_" + database + "_" + table; }
ASTPtr clone() const override
{
std::shared_ptr<ASTWatchQuery> res = std::make_shared<ASTWatchQuery>(*this);
res->children.clear();
cloneOutputOptions(*res);
return res;
}
protected:
void formatQueryImpl(const FormatSettings & s, FormatState & state, FormatStateStacked frame) const override
{
std::string indent_str = s.one_line ? "" : std::string(4 * frame.indent, ' ');
s.ostr << (s.hilite ? hilite_keyword : "") << "WATCH" << " " << (s.hilite ? hilite_none : "")
<< (!database.empty() ? backQuoteIfNeed(database) + "." : "") << backQuoteIfNeed(table);
if (is_watch_events)
{
s.ostr << " " << (s.hilite ? hilite_keyword : "") << "EVENTS" << (s.hilite ? hilite_none : "");
}
if (limit_length)
{
s.ostr << (s.hilite ? hilite_keyword : "") << s.nl_or_ws << indent_str << "LIMIT " << (s.hilite ? hilite_none : "");
limit_length->formatImpl(s, state, frame);
}
}
};
}

View File

@ -34,6 +34,13 @@ bool ParserAlterCommand::parseImpl(Pos & pos, ASTPtr & node, Expected & expected
ParserKeyword s_clear_index("CLEAR INDEX");
ParserKeyword s_materialize_index("MATERIALIZE INDEX");
ParserKeyword s_add("ADD");
ParserKeyword s_drop("DROP");
ParserKeyword s_suspend("SUSPEND");
ParserKeyword s_resume("RESUME");
ParserKeyword s_refresh("REFRESH");
ParserKeyword s_modify("MODIFY");
ParserKeyword s_attach_partition("ATTACH PARTITION");
ParserKeyword s_detach_partition("DETACH PARTITION");
ParserKeyword s_drop_partition("DROP PARTITION");
@ -65,268 +72,328 @@ bool ParserAlterCommand::parseImpl(Pos & pos, ASTPtr & node, Expected & expected
ParserList parser_assignment_list(
std::make_unique<ParserAssignment>(), std::make_unique<ParserToken>(TokenType::Comma),
/* allow_empty = */ false);
ParserNameList values_p;
if (s_add_column.ignore(pos, expected))
if (is_live_view)
{
if (s_if_not_exists.ignore(pos, expected))
command->if_not_exists = true;
if (!parser_col_decl.parse(pos, command->col_decl, expected))
return false;
if (s_after.ignore(pos, expected))
if (s_refresh.ignore(pos, expected))
{
command->type = ASTAlterCommand::LIVE_VIEW_REFRESH;
}
else
return false;
}
else if (is_live_channel)
{
if (s_add.ignore(pos, expected))
{
if (!values_p.parse(pos, command->values, expected))
return false;
command->type = ASTAlterCommand::LIVE_CHANNEL_ADD;
}
else if (s_drop.ignore(pos, expected))
{
if (!values_p.parse(pos, command->values, expected))
return false;
command->type = ASTAlterCommand::LIVE_CHANNEL_DROP;
}
else if (s_suspend.ignore(pos, expected))
{
if (!values_p.parse(pos, command->values, expected))
return false;
command->type = ASTAlterCommand::LIVE_CHANNEL_SUSPEND;
}
else if (s_resume.ignore(pos, expected))
{
if (!values_p.parse(pos, command->values, expected))
return false;
command->type = ASTAlterCommand::LIVE_CHANNEL_RESUME;
}
else if (s_refresh.ignore(pos, expected))
{
if (!values_p.parse(pos, command->values, expected))
return false;
command->type = ASTAlterCommand::LIVE_CHANNEL_REFRESH;
}
else if (s_modify.ignore(pos, expected))
{
if (!values_p.parse(pos, command->values, expected))
return false;
command->type = ASTAlterCommand::LIVE_CHANNEL_MODIFY;
}
else
return false;
}
else
{
if (s_add_column.ignore(pos, expected))
{
if (s_if_not_exists.ignore(pos, expected))
command->if_not_exists = true;
if (!parser_col_decl.parse(pos, command->col_decl, expected))
return false;
if (s_after.ignore(pos, expected))
{
if (!parser_name.parse(pos, command->column, expected))
return false;
}
command->type = ASTAlterCommand::ADD_COLUMN;
}
else if (s_drop_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
command->type = ASTAlterCommand::DROP_PARTITION;
}
else if (s_drop_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->column, expected))
return false;
command->type = ASTAlterCommand::DROP_COLUMN;
command->detach = false;
}
command->type = ASTAlterCommand::ADD_COLUMN;
}
else if (s_drop_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
command->type = ASTAlterCommand::DROP_PARTITION;
}
else if (s_drop_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->column, expected))
return false;
command->type = ASTAlterCommand::DROP_COLUMN;
command->detach = false;
}
else if (s_clear_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->column, expected))
return false;
command->type = ASTAlterCommand::DROP_COLUMN;
command->clear_column = true;
command->detach = false;
if (s_in_partition.ignore(pos, expected))
else if (s_clear_column.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->column, expected))
return false;
command->type = ASTAlterCommand::DROP_COLUMN;
command->clear_column = true;
command->detach = false;
if (s_in_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
}
}
}
else if (s_add_index.ignore(pos, expected))
{
if (s_if_not_exists.ignore(pos, expected))
command->if_not_exists = true;
if (!parser_idx_decl.parse(pos, command->index_decl, expected))
return false;
if (s_after.ignore(pos, expected))
else if (s_add_index.ignore(pos, expected))
{
if (s_if_not_exists.ignore(pos, expected))
command->if_not_exists = true;
if (!parser_idx_decl.parse(pos, command->index_decl, expected))
return false;
if (s_after.ignore(pos, expected))
{
if (!parser_name.parse(pos, command->index, expected))
return false;
}
command->type = ASTAlterCommand::ADD_INDEX;
}
else if (s_drop_index.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->index, expected))
return false;
command->type = ASTAlterCommand::DROP_INDEX;
command->detach = false;
}
command->type = ASTAlterCommand::ADD_INDEX;
}
else if (s_drop_index.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->index, expected))
return false;
command->type = ASTAlterCommand::DROP_INDEX;
command->detach = false;
}
else if (s_clear_index.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->index, expected))
return false;
command->type = ASTAlterCommand::DROP_INDEX;
command->clear_index = true;
command->detach = false;
if (!s_in_partition.ignore(pos, expected))
return false;
if (!parser_partition.parse(pos, command->partition, expected))
return false;
}
else if (s_materialize_index.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->index, expected))
return false;
command->type = ASTAlterCommand::MATERIALIZE_INDEX;
command->detach = false;
if (s_in_partition.ignore(pos, expected))
else if (s_clear_index.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->index, expected))
return false;
command->type = ASTAlterCommand::DROP_INDEX;
command->clear_index = true;
command->detach = false;
if (!s_in_partition.ignore(pos, expected))
return false;
if (!parser_partition.parse(pos, command->partition, expected))
return false;
}
}
else if (s_detach_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
command->type = ASTAlterCommand::DROP_PARTITION;
command->detach = true;
}
else if (s_attach_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
if (s_from.ignore(pos))
else if (s_materialize_index.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->index, expected))
return false;
command->type = ASTAlterCommand::MATERIALIZE_INDEX;
command->detach = false;
if (s_in_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
}
}
else if (s_detach_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
command->type = ASTAlterCommand::DROP_PARTITION;
command->detach = true;
}
else if (s_attach_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
if (s_from.ignore(pos))
{
if (!parseDatabaseAndTableName(pos, expected, command->from_database, command->from_table))
return false;
command->replace = false;
command->type = ASTAlterCommand::REPLACE_PARTITION;
}
else
{
command->type = ASTAlterCommand::ATTACH_PARTITION;
}
}
else if (s_replace_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
if (!s_from.ignore(pos, expected))
return false;
if (!parseDatabaseAndTableName(pos, expected, command->from_database, command->from_table))
return false;
command->replace = false;
command->replace = true;
command->type = ASTAlterCommand::REPLACE_PARTITION;
}
else
else if (s_attach_part.ignore(pos, expected))
{
if (!parser_string_literal.parse(pos, command->partition, expected))
return false;
command->part = true;
command->type = ASTAlterCommand::ATTACH_PARTITION;
}
}
else if (s_replace_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
if (!s_from.ignore(pos, expected))
return false;
if (!parseDatabaseAndTableName(pos, expected, command->from_database, command->from_table))
return false;
command->replace = true;
command->type = ASTAlterCommand::REPLACE_PARTITION;
}
else if (s_attach_part.ignore(pos, expected))
{
if (!parser_string_literal.parse(pos, command->partition, expected))
return false;
command->part = true;
command->type = ASTAlterCommand::ATTACH_PARTITION;
}
else if (s_fetch_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
if (!s_from.ignore(pos, expected))
return false;
ASTPtr ast_from;
if (!parser_string_literal.parse(pos, ast_from, expected))
return false;
command->from = ast_from->as<ASTLiteral &>().value.get<const String &>();
command->type = ASTAlterCommand::FETCH_PARTITION;
}
else if (s_freeze.ignore(pos, expected))
{
if (s_partition.ignore(pos, expected))
else if (s_fetch_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
command->type = ASTAlterCommand::FREEZE_PARTITION;
if (!s_from.ignore(pos, expected))
return false;
ASTPtr ast_from;
if (!parser_string_literal.parse(pos, ast_from, expected))
return false;
command->from = ast_from->as<ASTLiteral &>().value.get<const String &>();
command->type = ASTAlterCommand::FETCH_PARTITION;
}
else if (s_freeze.ignore(pos, expected))
{
if (s_partition.ignore(pos, expected))
{
if (!parser_partition.parse(pos, command->partition, expected))
return false;
command->type = ASTAlterCommand::FREEZE_PARTITION;
}
else
{
command->type = ASTAlterCommand::FREEZE_ALL;
}
/// WITH NAME 'name' - place local backup to directory with specified name
if (s_with.ignore(pos, expected))
{
if (!s_name.ignore(pos, expected))
return false;
ASTPtr ast_with_name;
if (!parser_string_literal.parse(pos, ast_with_name, expected))
return false;
command->with_name = ast_with_name->as<ASTLiteral &>().value.get<const String &>();
}
}
else if (s_modify_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_modify_col_decl.parse(pos, command->col_decl, expected))
return false;
command->type = ASTAlterCommand::MODIFY_COLUMN;
}
else if (s_modify_order_by.ignore(pos, expected))
{
if (!parser_exp_elem.parse(pos, command->order_by, expected))
return false;
command->type = ASTAlterCommand::MODIFY_ORDER_BY;
}
else if (s_delete_where.ignore(pos, expected))
{
if (!parser_exp_elem.parse(pos, command->predicate, expected))
return false;
command->type = ASTAlterCommand::DELETE;
}
else if (s_update.ignore(pos, expected))
{
if (!parser_assignment_list.parse(pos, command->update_assignments, expected))
return false;
if (!s_where.ignore(pos, expected))
return false;
if (!parser_exp_elem.parse(pos, command->predicate, expected))
return false;
command->type = ASTAlterCommand::UPDATE;
}
else if (s_comment_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->column, expected))
return false;
if (!parser_string_literal.parse(pos, command->comment, expected))
return false;
command->type = ASTAlterCommand::COMMENT_COLUMN;
}
else if (s_modify_ttl.ignore(pos, expected))
{
if (!parser_exp_elem.parse(pos, command->ttl, expected))
return false;
command->type = ASTAlterCommand::MODIFY_TTL;
}
else
{
command->type = ASTAlterCommand::FREEZE_ALL;
}
/// WITH NAME 'name' - place local backup to directory with specified name
if (s_with.ignore(pos, expected))
{
if (!s_name.ignore(pos, expected))
return false;
ASTPtr ast_with_name;
if (!parser_string_literal.parse(pos, ast_with_name, expected))
return false;
command->with_name = ast_with_name->as<ASTLiteral &>().value.get<const String &>();
}
return false;
}
else if (s_modify_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_modify_col_decl.parse(pos, command->col_decl, expected))
return false;
command->type = ASTAlterCommand::MODIFY_COLUMN;
}
else if (s_modify_order_by.ignore(pos, expected))
{
if (!parser_exp_elem.parse(pos, command->order_by, expected))
return false;
command->type = ASTAlterCommand::MODIFY_ORDER_BY;
}
else if (s_delete_where.ignore(pos, expected))
{
if (!parser_exp_elem.parse(pos, command->predicate, expected))
return false;
command->type = ASTAlterCommand::DELETE;
}
else if (s_update.ignore(pos, expected))
{
if (!parser_assignment_list.parse(pos, command->update_assignments, expected))
return false;
if (!s_where.ignore(pos, expected))
return false;
if (!parser_exp_elem.parse(pos, command->predicate, expected))
return false;
command->type = ASTAlterCommand::UPDATE;
}
else if (s_comment_column.ignore(pos, expected))
{
if (s_if_exists.ignore(pos, expected))
command->if_exists = true;
if (!parser_name.parse(pos, command->column, expected))
return false;
if (!parser_string_literal.parse(pos, command->comment, expected))
return false;
command->type = ASTAlterCommand::COMMENT_COLUMN;
}
else if (s_modify_ttl.ignore(pos, expected))
{
if (!parser_exp_elem.parse(pos, command->ttl, expected))
return false;
command->type = ASTAlterCommand::MODIFY_TTL;
}
else
return false;
if (command->col_decl)
command->children.push_back(command->col_decl);
@ -340,6 +407,8 @@ bool ParserAlterCommand::parseImpl(Pos & pos, ASTPtr & node, Expected & expected
command->children.push_back(command->predicate);
if (command->update_assignments)
command->children.push_back(command->update_assignments);
if (command->values)
command->children.push_back(command->values);
if (command->comment)
command->children.push_back(command->comment);
if (command->ttl)
@ -355,7 +424,7 @@ bool ParserAlterCommandList::parseImpl(Pos & pos, ASTPtr & node, Expected & expe
node = command_list;
ParserToken s_comma(TokenType::Comma);
ParserAlterCommand p_command;
ParserAlterCommand p_command(is_live_view, is_live_channel);
do
{
@ -404,8 +473,30 @@ bool ParserAlterQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
node = query;
ParserKeyword s_alter_table("ALTER TABLE");
ParserKeyword s_alter_live_view("ALTER LIVE VIEW");
ParserKeyword s_alter_live_channel("ALTER LIVE CHANNEL");
bool is_live_view = false;
bool is_live_channel = false;
if (!s_alter_table.ignore(pos, expected))
return false;
{
if (!s_alter_live_view.ignore(pos, expected))
{
if (!s_alter_live_channel.ignore(pos, expected))
return false;
else
is_live_channel = true;
}
else
is_live_view = true;
}
if (is_live_view)
query->is_live_view = true;
if (is_live_channel)
query->is_live_channel = true;
if (!parseDatabaseAndTableName(pos, expected, query->database, query->table))
return false;
@ -418,7 +509,7 @@ bool ParserAlterQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
}
query->cluster = cluster_str;
ParserAlterCommandList p_command_list;
ParserAlterCommandList p_command_list(is_live_view, is_live_channel);
ASTPtr command_list;
if (!p_command_list.parse(pos, command_list, expected))
return false;

View File

@ -19,6 +19,15 @@ namespace DB
* [FREEZE [PARTITION] [WITH NAME name]]
* [DELETE WHERE ...]
* [UPDATE col_name = expr, ... WHERE ...]
* ALTER LIVE VIEW [db.name]
* [REFRESH]
* ALTER LIVE CHANNEL [db.name] [ON CLUSTER cluster]
* [ADD live_view, ...]
* [DROP live_view, ...]
* [SUSPEND live_view, ...]
* [RESUME live_view, ...]
* [REFRESH live_view, ...]
* [MODIFY live_view, ...]
*/
class ParserAlterQuery : public IParserBase
@ -34,6 +43,12 @@ class ParserAlterCommandList : public IParserBase
protected:
const char * getName() const { return "a list of ALTER commands"; }
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected);
public:
bool is_live_view;
bool is_live_channel;
ParserAlterCommandList(bool is_live_view_ = false, bool is_live_channel_ = false) : is_live_view(is_live_view_), is_live_channel(is_live_channel_) {}
};
@ -42,6 +57,12 @@ class ParserAlterCommand : public IParserBase
protected:
const char * getName() const { return "ALTER command"; }
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected);
public:
bool is_live_view;
bool is_live_channel;
ParserAlterCommand(bool is_live_view_ = false, bool is_live_channel_ = false) : is_live_view(is_live_view_), is_live_channel(is_live_channel_) {}
};

View File

@ -94,6 +94,12 @@ bool ParserColumnDeclarationList::parseImpl(Pos & pos, ASTPtr & node, Expected &
.parse(pos, node, expected);
}
bool ParserNameList::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
{
return ParserList(std::make_unique<ParserCompoundIdentifier>(), std::make_unique<ParserToken>(TokenType::Comma), false)
.parse(pos, node, expected);
}
bool ParserIndexDeclaration::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
{
ParserKeyword s_type("TYPE");
@ -309,7 +315,10 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
ParserKeyword s_if_not_exists("IF NOT EXISTS");
ParserKeyword s_as("AS");
ParserKeyword s_view("VIEW");
ParserKeyword s_with("WITH");
ParserKeyword s_materialized("MATERIALIZED");
ParserKeyword s_live("LIVE");
ParserKeyword s_channel("CHANNEL");
ParserKeyword s_populate("POPULATE");
ParserKeyword s_or_replace("OR REPLACE");
ParserToken s_dot(TokenType::Dot);
@ -320,6 +329,7 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
ParserColumnsOrIndicesDeclarationList columns_or_indices_p;
ParserSelectWithUnionQuery select_p;
ParserFunction table_function_p;
ParserNameList names_p;
ASTPtr database;
ASTPtr table;
@ -331,11 +341,15 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
ASTPtr as_table;
ASTPtr as_table_function;
ASTPtr select;
ASTPtr tables;
String cluster_str;
bool attach = false;
bool if_not_exists = false;
bool is_view = false;
bool is_materialized_view = false;
bool is_live_view = false;
bool is_live_channel = false;
bool is_populate = false;
bool is_temporary = false;
bool replace_view = false;
@ -431,6 +445,79 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
}
}
}
else if (s_live.ignore(pos, expected))
{
if (s_channel.ignore(pos, expected))
is_live_channel = true;
else if (s_view.ignore(pos, expected))
is_live_view = true;
else
return false;
if (s_if_not_exists.ignore(pos, expected))
if_not_exists = true;
if (!name_p.parse(pos, table, expected))
return false;
if (s_dot.ignore(pos, expected))
{
database = table;
if (!name_p.parse(pos, table, expected))
return false;
}
if (ParserKeyword{"ON"}.ignore(pos, expected))
{
if (!ASTQueryWithOnCluster::parse(pos, cluster_str, expected))
return false;
}
if (!is_live_channel)
{
// TO [db.]table
if (ParserKeyword{"TO"}.ignore(pos, expected))
{
if (!name_p.parse(pos, to_table, expected))
return false;
if (s_dot.ignore(pos, expected))
{
to_database = to_table;
if (!name_p.parse(pos, to_table, expected))
return false;
}
}
}
/// Optional - a list of columns can be specified. It must fully comply with SELECT.
if (s_lparen.ignore(pos, expected))
{
if (!columns_or_indices_p.parse(pos, columns_list, expected))
return false;
if (!s_rparen.ignore(pos, expected))
return false;
}
if (is_live_channel)
{
if (s_with.ignore(pos, expected))
{
if (!names_p.parse(pos, tables, expected))
return false;
}
}
else
{
/// AS SELECT ...
if (!s_as.ignore(pos, expected))
return false;
if (!select_p.parse(pos, select, expected))
return false;
}
}
else if (is_temporary)
return false;
else if (s_database.ignore(pos, expected))
@ -538,6 +625,8 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
query->if_not_exists = if_not_exists;
query->is_view = is_view;
query->is_materialized_view = is_materialized_view;
query->is_live_view = is_live_view;
query->is_live_channel = is_live_channel;
query->is_populate = is_populate;
query->temporary = is_temporary;
query->replace_view = replace_view;
@ -551,6 +640,7 @@ bool ParserCreateQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
query->set(query->columns_list, columns_list);
query->set(query->storage, storage);
query->set(query->tables, tables);
tryGetIdentifierNameInto(as_database, query->as_database);
tryGetIdentifierNameInto(as_table, query->as_table);

View File

@ -91,6 +91,14 @@ protected:
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected);
};
/** List of table names. */
class ParserNameList : public IParserBase
{
protected:
const char * getName() const { return "name list"; }
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected);
};
template <typename NameParser>
class IParserColumnDeclaration : public IParserBase
@ -300,7 +308,7 @@ protected:
* CREATE|ATTACH DATABASE db [ENGINE = engine]
*
* Or:
* CREATE [OR REPLACE]|ATTACH [MATERIALIZED] VIEW [IF NOT EXISTS] [db.]name [TO [db.]name] [ENGINE = engine] [POPULATE] AS SELECT ...
* CREATE[OR REPLACE]|ATTACH [[MATERIALIZED] VIEW] | [[TEMPORARY] LIVE [CHANNEL] | [VIEW]] [IF NOT EXISTS] [db.]name [TO [db.]name] [ENGINE = engine] [POPULATE] AS SELECT ...
*/
class ParserCreateQuery : public IParserBase
{

View File

@ -11,6 +11,7 @@
#include <Parsers/ParserDropQuery.h>
#include <Parsers/ParserKillQueryQuery.h>
#include <Parsers/ParserOptimizeQuery.h>
#include <Parsers/ParserWatchQuery.h>
#include <Parsers/ParserSetQuery.h>
#include <Parsers/ASTExplainQuery.h>
@ -32,6 +33,7 @@ bool ParserQueryWithOutput::parseImpl(Pos & pos, ASTPtr & node, Expected & expec
ParserCheckQuery check_p;
ParserOptimizeQuery optimize_p;
ParserKillQueryQuery kill_query_p;
ParserWatchQuery watch_p;
ASTPtr query;
@ -57,7 +59,8 @@ bool ParserQueryWithOutput::parseImpl(Pos & pos, ASTPtr & node, Expected & expec
|| drop_p.parse(pos, query, expected)
|| check_p.parse(pos, query, expected)
|| kill_query_p.parse(pos, query, expected)
|| optimize_p.parse(pos, query, expected);
|| optimize_p.parse(pos, query, expected)
|| watch_p.parse(pos, query, expected);
if (!parsed)
return false;

View File

@ -0,0 +1,77 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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 <Parsers/ASTLiteral.h>
#include <Parsers/ASTIdentifier.h>
#include <Parsers/ASTWatchQuery.h>
#include <Parsers/CommonParsers.h>
#include <Parsers/ParserWatchQuery.h>
#include <Parsers/ExpressionElementParsers.h>
namespace DB
{
bool ParserWatchQuery::parseImpl(Pos & pos, ASTPtr & node, Expected & expected)
{
ParserKeyword s_watch("WATCH");
ParserToken s_dot(TokenType::Dot);
ParserIdentifier name_p;
ParserKeyword s_events("EVENTS");
ParserKeyword s_limit("LIMIT");
ASTPtr database;
ASTPtr table;
auto query = std::make_shared<ASTWatchQuery>();
if (!s_watch.ignore(pos, expected))
{
return false;
}
if (!name_p.parse(pos, table, expected))
return false;
if (s_dot.ignore(pos, expected))
{
database = table;
if (!name_p.parse(pos, table, expected))
return false;
}
/// EVENTS
if (s_events.ignore(pos, expected))
{
query->is_watch_events = true;
}
/// LIMIT length
if (s_limit.ignore(pos, expected))
{
ParserNumber num;
if (!num.parse(pos, query->limit_length, expected))
return false;
}
if (database)
query->database = getIdentifierName(database);
if (table)
query->table = getIdentifierName(table);
node = query;
return true;
}
}

View File

@ -0,0 +1,30 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <Parsers/IParserBase.h>
namespace DB
{
/** Query like this:
* WATCH [db.]table EVENTS
*/
class ParserWatchQuery : public IParserBase
{
protected:
const char * getName() const { return "WATCH query"; }
bool parseImpl(Pos & pos, ASTPtr & node, Expected & expected);
};
}

View File

@ -29,8 +29,9 @@ protected:
void consumeTotals(Chunk) override {}
void consumeExtremes(Chunk) override {}
private:
size_t field_number = 0;
private:
Names fields;
FormatSettings settings;

View File

@ -0,0 +1,44 @@
#include <IO/WriteHelpers.h>
#include <IO/WriteBufferValidUTF8.h>
#include <Processors/Formats/Impl/JSONEachRowWithProgressRowOutputFormat.h>
#include <Formats/FormatFactory.h>
namespace DB
{
void JSONEachRowWithProgressRowOutputFormat::writeRowStartDelimiter()
{
writeCString("{\"row\":{", out);
}
void JSONEachRowWithProgressRowOutputFormat::writeRowEndDelimiter()
{
writeCString("}}\n", out);
field_number = 0;
}
void JSONEachRowWithProgressRowOutputFormat::onProgress(const Progress & value)
{
progress.incrementPiecewiseAtomically(value);
writeCString("{\"progress\":", out);
progress.writeJSON(out);
writeCString("}\n", out);
}
void registerOutputFormatProcessorJSONEachRowWithProgress(FormatFactory & factory)
{
factory.registerOutputFormatProcessor("JSONEachRowWithProgress", [](
WriteBuffer & buf,
const Block & sample,
const Context &,
FormatFactory::WriteCallback callback,
const FormatSettings & format_settings)
{
return std::make_shared<JSONEachRowWithProgressRowOutputFormat>(buf, sample, callback, format_settings);
});
}
}

View File

@ -0,0 +1,20 @@
#pragma once
#include <Processors/Formats/Impl/JSONEachRowRowOutputFormat.h>
namespace DB
{
class JSONEachRowWithProgressRowOutputFormat : public JSONEachRowRowOutputFormat
{
public:
using JSONEachRowRowOutputFormat::JSONEachRowRowOutputFormat;
void writeRowStartDelimiter() override;
void writeRowEndDelimiter() override;
void onProgress(const Progress & value) override;
private:
Progress progress;
};
}

View File

@ -111,7 +111,7 @@ static void fillColumnWithStringData(std::shared_ptr<arrow::Column> & arrow_colu
static void fillColumnWithBooleanData(std::shared_ptr<arrow::Column> & arrow_column, MutableColumnPtr & internal_column)
{
auto & column_data = static_cast<ColumnVector<UInt8> &>(*internal_column).getData();
column_data.resize(arrow_column->length());
column_data.reserve(arrow_column->length());
for (size_t chunk_i = 0, num_chunks = static_cast<size_t>(arrow_column->data()->num_chunks()); chunk_i < num_chunks; ++chunk_i)
{
@ -120,7 +120,7 @@ static void fillColumnWithBooleanData(std::shared_ptr<arrow::Column> & arrow_col
std::shared_ptr<arrow::Buffer> buffer = chunk.data()->buffers[1];
for (size_t bool_i = 0; bool_i != static_cast<size_t>(chunk.length()); ++bool_i)
column_data[bool_i] = chunk.Value(bool_i);
column_data.emplace_back(chunk.Value(bool_i));
}
}

View File

@ -76,7 +76,7 @@ public:
/// The name of the table.
virtual std::string getTableName() const = 0;
virtual std::string getDatabaseName() const = 0;
virtual std::string getDatabaseName() const { return {}; }
/// Returns true if the storage receives data from a remote server or servers.
virtual bool isRemote() const { return false; }
@ -102,7 +102,8 @@ public:
virtual ColumnSizeByName getColumnSizes() const { return {}; }
public: /// thread-unsafe part. lockStructure must be acquired
const ColumnsDescription & getColumns() const; /// returns combined set of columns
virtual const ColumnsDescription & getColumns() const; /// returns combined set of columns
virtual void setColumns(ColumnsDescription columns_); /// sets only real columns, possibly overwrites virtual ones.
const ColumnsDescription & getVirtuals() const;
const IndicesDescription & getIndices() const;
@ -132,7 +133,6 @@ public: /// thread-unsafe part. lockStructure must be acquired
void check(const Block & block, bool need_all = false) const;
protected: /// still thread-unsafe part.
void setColumns(ColumnsDescription columns_); /// sets only real columns, possibly overwrites virtual ones.
void setIndices(IndicesDescription indices_);
/// Returns whether the column is virtual - by default all columns are real.
@ -172,6 +172,36 @@ public:
*/
virtual QueryProcessingStage::Enum getQueryProcessingStage(const Context &) const { return QueryProcessingStage::FetchColumns; }
/** Watch live changes to the table.
* Accepts a list of columns to read, as well as a description of the query,
* from which information can be extracted about how to retrieve data
* (indexes, locks, etc.)
* Returns a stream with which you can read data sequentially
* or multiple streams for parallel data reading.
* The `processed_stage` info is also written to what stage the request was processed.
* (Normally, the function only reads the columns from the list, but in other cases,
* for example, the request can be partially processed on a remote server.)
*
* context contains settings for one query.
* Usually Storage does not care about these settings, since they are used in the interpreter.
* But, for example, for distributed query processing, the settings are passed to the remote server.
*
* num_streams - a recommendation, how many streams to return,
* if the storage can return a different number of streams.
*
* It is guaranteed that the structure of the table will not change over the lifetime of the returned streams (that is, there will not be ALTER, RENAME and DROP).
*/
virtual BlockInputStreams watch(
const Names & /*column_names*/,
const SelectQueryInfo & /*query_info*/,
const Context & /*context*/,
QueryProcessingStage::Enum & /*processed_stage*/,
size_t /*max_block_size*/,
unsigned /*num_streams*/)
{
throw Exception("Method watch is not supported by storage " + getName(), ErrorCodes::NOT_IMPLEMENTED);
}
/** Read a set of columns from the table.
* Accepts a list of columns to read, as well as a description of the query,
* from which information can be extracted about how to retrieve data
@ -296,7 +326,7 @@ public:
return {};
}
bool is_dropped{false};
std::atomic<bool> is_dropped{false};
/// Does table support index for IN sections
virtual bool supportsIndexForIn() const { return false; }

View File

@ -0,0 +1,65 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <Parsers/ASTAlterQuery.h>
#include <optional>
#include <Storages/StorageLiveView.h>
namespace DB
{
namespace ErrorCodes
{
extern const int UNKNOWN_STORAGE;
}
struct LiveViewCommand
{
enum Type
{
REFRESH
};
Type type;
ASTPtr values;
static LiveViewCommand refresh(const ASTPtr & values)
{
LiveViewCommand res;
res.type = REFRESH;
res.values = values;
return res;
}
static std::optional<LiveViewCommand> parse(ASTAlterCommand * command)
{
if (command->type == ASTAlterCommand::LIVE_VIEW_REFRESH)
return refresh(command->values);
return {};
}
};
class LiveViewCommands : public std::vector<LiveViewCommand>
{
public:
void validate(const IStorage & table)
{
if (!empty() && !dynamic_cast<const StorageLiveView *>(&table))
throw Exception("Wrong storage type. Must be StorageLiveView", DB::ErrorCodes::UNKNOWN_STORAGE);
}
};
}

View File

@ -521,7 +521,7 @@ public:
/// parts should be sorted.
MergeTreeData::MutableDataPartPtr MergeTreeDataMergerMutator::mergePartsToTemporaryPart(
const FutureMergedMutatedPart & future_part, MergeList::Entry & merge_entry,
const FutureMergedMutatedPart & future_part, MergeList::Entry & merge_entry, TableStructureReadLockHolder &,
time_t time_of_merge, DiskSpaceMonitor::Reservation * disk_reservation, bool deduplicate, bool force_ttl)
{
static const String TMP_PREFIX = "tmp_merge_";
@ -883,7 +883,8 @@ MergeTreeData::MutableDataPartPtr MergeTreeDataMergerMutator::mutatePartToTempor
const FutureMergedMutatedPart & future_part,
const std::vector<MutationCommand> & commands,
MergeListEntry & merge_entry,
const Context & context)
const Context & context,
TableStructureReadLockHolder & table_lock_holder)
{
auto check_not_cancelled = [&]()
{
@ -918,7 +919,6 @@ MergeTreeData::MutableDataPartPtr MergeTreeDataMergerMutator::mutatePartToTempor
command.partition, context_for_reading);
});
MutationsInterpreter mutations_interpreter(storage_from_source_part, commands_for_part, context_for_reading);
if (!mutations_interpreter.isStorageTouchedByMutations())
@ -949,7 +949,7 @@ MergeTreeData::MutableDataPartPtr MergeTreeDataMergerMutator::mutatePartToTempor
Poco::File(new_part_tmp_path).createDirectories();
auto in = mutations_interpreter.execute();
auto in = mutations_interpreter.execute(table_lock_holder);
const auto & updated_header = mutations_interpreter.getUpdatedHeader();
NamesAndTypesList all_columns = data.getColumns().getAllPhysical();

View File

@ -94,14 +94,14 @@ public:
*/
MergeTreeData::MutableDataPartPtr mergePartsToTemporaryPart(
const FutureMergedMutatedPart & future_part,
MergeListEntry & merge_entry, time_t time_of_merge,
MergeListEntry & merge_entry, TableStructureReadLockHolder & table_lock_holder, time_t time_of_merge,
DiskSpaceMonitor::Reservation * disk_reservation, bool deduplication, bool force_ttl);
/// Mutate a single data part with the specified commands. Will create and return a temporary part.
MergeTreeData::MutableDataPartPtr mutatePartToTemporaryPart(
const FutureMergedMutatedPart & future_part,
const std::vector<MutationCommand> & commands,
MergeListEntry & merge_entry, const Context & context);
MergeListEntry & merge_entry, const Context & context, TableStructureReadLockHolder & table_lock_holder);
MergeTreeData::DataPartPtr renameMergedTemporaryPart(
MergeTreeData::MutableDataPartPtr & new_data_part,

View File

@ -1281,9 +1281,9 @@ bool ReplicatedMergeTreeQueue::tryFinalizeMutations(zkutil::ZooKeeperPtr zookeep
}
if (!finished.empty())
{
zookeeper->set(replica_path + "/mutation_pointer", finished.back()->znode_name);
{
std::lock_guard lock(state_mutex);
mutation_pointer = finished.back()->znode_name;
@ -1655,7 +1655,7 @@ bool ReplicatedMergeTreeMergePredicate::operator()(
std::optional<Int64> ReplicatedMergeTreeMergePredicate::getDesiredMutationVersion(const MergeTreeData::DataPartPtr & part) const
{
/// Assigning mutations is easier than assigning merges because mutations appear in the same order as
/// the order of their version numbers (see StorageReplicatedMergeTree::mutate()).
/// the order of their version numbers (see StorageReplicatedMergeTree::mutate).
/// This means that if we have loaded the mutation with version number X then all mutations with
/// the version numbers less than X are also loaded and if there is no merge or mutation assigned to
/// the part (checked by querying queue.virtual_parts), we can confidently assign a mutation to

View File

@ -0,0 +1,66 @@
#pragma once
#include <Storages/IStorage.h>
namespace DB
{
class ProxyStorage : public IStorage
{
public:
ProxyStorage(StoragePtr storage_, BlockInputStreams streams_, QueryProcessingStage::Enum to_stage_)
: storage(std::move(storage_)), streams(std::move(streams_)), to_stage(to_stage_) {}
public:
std::string getName() const override { return "ProxyStorage(" + storage->getName() + ")"; }
std::string getTableName() const override { return storage->getTableName(); }
bool isRemote() const override { return storage->isRemote(); }
bool supportsSampling() const override { return storage->supportsSampling(); }
bool supportsFinal() const override { return storage->supportsFinal(); }
bool supportsPrewhere() const override { return storage->supportsPrewhere(); }
bool supportsReplication() const override { return storage->supportsReplication(); }
bool supportsDeduplication() const override { return storage->supportsDeduplication(); }
QueryProcessingStage::Enum getQueryProcessingStage(const Context & /*context*/) const override { return to_stage; }
BlockInputStreams read(
const Names & /*column_names*/,
const SelectQueryInfo & /*query_info*/,
const Context & /*context*/,
QueryProcessingStage::Enum /*processed_stage*/,
size_t /*max_block_size*/,
unsigned /*num_streams*/) override
{
return streams;
}
bool supportsIndexForIn() const override { return storage->supportsIndexForIn(); }
bool mayBenefitFromIndexForIn(const ASTPtr & left_in_operand, const Context & query_context) const override { return storage->mayBenefitFromIndexForIn(left_in_operand, query_context); }
ASTPtr getPartitionKeyAST() const override { return storage->getPartitionKeyAST(); }
ASTPtr getSortingKeyAST() const override { return storage->getSortingKeyAST(); }
ASTPtr getPrimaryKeyAST() const override { return storage->getPrimaryKeyAST(); }
ASTPtr getSamplingKeyAST() const override { return storage->getSamplingKeyAST(); }
Names getColumnsRequiredForPartitionKey() const override { return storage->getColumnsRequiredForPartitionKey(); }
Names getColumnsRequiredForSortingKey() const override { return storage->getColumnsRequiredForSortingKey(); }
Names getColumnsRequiredForPrimaryKey() const override { return storage->getColumnsRequiredForPrimaryKey(); }
Names getColumnsRequiredForSampling() const override { return storage->getColumnsRequiredForSampling(); }
Names getColumnsRequiredForFinal() const override { return storage->getColumnsRequiredForFinal(); }
const ColumnsDescription & getColumns() const override { return storage->getColumns(); }
void setColumns(ColumnsDescription columns_) override { return storage->setColumns(columns_); }
NameAndTypePair getColumn(const String & column_name) const override { return storage->getColumn(column_name); }
bool hasColumn(const String & column_name) const override { return storage->hasColumn(column_name); }
static StoragePtr createProxyStorage(StoragePtr storage, BlockInputStreams streams, QueryProcessingStage::Enum to_stage)
{
return std::make_shared<ProxyStorage>(std::move(storage), std::move(streams), to_stage);
}
private:
StoragePtr storage;
BlockInputStreams streams;
QueryProcessingStage::Enum to_stage;
};
}

View File

@ -317,8 +317,8 @@ BlockInputStreams StorageDistributed::read(
const auto & modified_query_ast = rewriteSelectQuery(
query_info.query, remote_database, remote_table, remote_table_function_ptr);
Block header = materializeBlock(
InterpreterSelectQuery(query_info.query, context, SelectQueryOptions(processed_stage)).getSampleBlock());
Block header =
InterpreterSelectQuery(query_info.query, context, SelectQueryOptions(processed_stage)).getSampleBlock();
ClusterProxy::SelectStreamFactory select_stream_factory = remote_table_function_ptr
? ClusterProxy::SelectStreamFactory(

View File

@ -60,6 +60,22 @@ StoragePtr StorageFactory::get(
name = "View";
}
else if (query.is_live_view)
{
if (query.storage)
throw Exception("Specifying ENGINE is not allowed for a LiveView", ErrorCodes::INCORRECT_QUERY);
name = "LiveView";
}
else if (query.is_live_channel)
{
if (query.storage)
throw Exception("Specifying ENGINE is not allowed for a LiveChannel", ErrorCodes::INCORRECT_QUERY);
name = "LiveChannel";
}
else
{
/// Check for some special types, that are not allowed to be stored in tables. Example: NULL data type.
@ -115,6 +131,18 @@ StoragePtr StorageFactory::get(
"Direct creation of tables with ENGINE MaterializedView is not supported, use CREATE MATERIALIZED VIEW statement",
ErrorCodes::INCORRECT_QUERY);
}
else if (name == "LiveView")
{
throw Exception(
"Direct creation of tables with ENGINE LiveView is not supported, use CREATE LIVE VIEW statement",
ErrorCodes::INCORRECT_QUERY);
}
else if (name == "LiveChannel")
{
throw Exception(
"Direct creation of tables with ENGINE LiveChannel is not supported, use CREATE LIVE CHANNEL statement",
ErrorCodes::INCORRECT_QUERY);
}
}
}

View File

@ -0,0 +1,479 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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 <Parsers/ASTSelectQuery.h>
#include <Parsers/ASTCreateQuery.h>
#include <Parsers/ASTWatchQuery.h>
#include <Parsers/ASTDropQuery.h>
#include <Parsers/ASTIdentifier.h>
#include <Parsers/ASTLiteral.h>
#include <Interpreters/Context.h>
#include <Interpreters/InterpreterDropQuery.h>
#include <DataStreams/NullBlockInputStream.h>
#include <DataStreams/LiveViewBlockInputStream.h>
#include <DataStreams/LiveViewEventsBlockInputStream.h>
#include <DataStreams/MaterializingBlockInputStream.h>
#include <Common/typeid_cast.h>
#include <Storages/StorageLiveView.h>
#include <Storages/StorageFactory.h>
#include <Storages/ProxyStorage.h>
#include <Parsers/ASTTablesInSelectQuery.h>
#include <Parsers/ASTSubquery.h>
#include <Interpreters/DatabaseAndTableWithAlias.h>
#include <Interpreters/AddDefaultDatabaseVisitor.h>
namespace DB
{
namespace ErrorCodes
{
extern const int LOGICAL_ERROR;
extern const int INCORRECT_QUERY;
extern const int TABLE_WAS_NOT_DROPPED;
extern const int QUERY_IS_NOT_SUPPORTED_IN_LIVE_VIEW;
}
static void extractDependentTable(ASTSelectQuery & query, String & select_database_name, String & select_table_name)
{
auto db_and_table = getDatabaseAndTable(query, 0);
ASTPtr subquery = extractTableExpression(query, 0);
if (!db_and_table && !subquery)
return;
if (db_and_table)
{
select_table_name = db_and_table->table;
if (db_and_table->database.empty())
{
db_and_table->database = select_database_name;
AddDefaultDatabaseVisitor visitor(select_database_name);
visitor.visit(query);
}
else
select_database_name = db_and_table->database;
}
else if (auto * ast_select = subquery->as<ASTSelectWithUnionQuery>())
{
if (ast_select->list_of_selects->children.size() != 1)
throw Exception("UNION is not supported for LIVE VIEW", ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_LIVE_VIEW);
auto & inner_query = ast_select->list_of_selects->children.at(0);
extractDependentTable(inner_query->as<ASTSelectQuery &>(), select_database_name, select_table_name);
}
else
throw Exception("Logical error while creating StorageLiveView."
" Could not retrieve table name from select query.",
DB::ErrorCodes::LOGICAL_ERROR);
}
static void checkAllowedQueries(const ASTSelectQuery & query)
{
if (query.prewhere() || query.final() || query.sample_size())
throw Exception("LIVE VIEW cannot have PREWHERE, SAMPLE or FINAL.", DB::ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_LIVE_VIEW);
ASTPtr subquery = extractTableExpression(query, 0);
if (!subquery)
return;
if (const auto * ast_select = subquery->as<ASTSelectWithUnionQuery>())
{
if (ast_select->list_of_selects->children.size() != 1)
throw Exception("UNION is not supported for LIVE VIEW", ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_LIVE_VIEW);
const auto & inner_query = ast_select->list_of_selects->children.at(0);
checkAllowedQueries(inner_query->as<ASTSelectQuery &>());
}
}
StorageLiveView::StorageLiveView(
const String & table_name_,
const String & database_name_,
Context & local_context,
const ASTCreateQuery & query,
const ColumnsDescription & columns_)
: IStorage(columns_), table_name(table_name_),
database_name(database_name_), global_context(local_context.getGlobalContext())
{
if (!query.select)
throw Exception("SELECT query is not specified for " + getName(), ErrorCodes::INCORRECT_QUERY);
/// Default value, if only table name exist in the query
select_database_name = local_context.getCurrentDatabase();
if (query.select->list_of_selects->children.size() != 1)
throw Exception("UNION is not supported for LIVE VIEW", ErrorCodes::QUERY_IS_NOT_SUPPORTED_IN_LIVE_VIEW);
inner_query = query.select->list_of_selects->children.at(0);
ASTSelectQuery & select_query = typeid_cast<ASTSelectQuery &>(*inner_query);
extractDependentTable(select_query, select_database_name, select_table_name);
/// If the table is not specified - use the table `system.one`
if (select_table_name.empty())
{
select_database_name = "system";
select_table_name = "one";
}
global_context.addDependency(
DatabaseAndTableName(select_database_name, select_table_name),
DatabaseAndTableName(database_name, table_name));
is_temporary = query.temporary;
temporary_live_view_timeout = local_context.getSettingsRef().temporary_live_view_timeout.totalSeconds();
blocks_ptr = std::make_shared<BlocksPtr>();
blocks_metadata_ptr = std::make_shared<BlocksMetadataPtr>();
active_ptr = std::make_shared<bool>(true);
}
NameAndTypePair StorageLiveView::getColumn(const String & column_name) const
{
if (column_name == "_version")
return NameAndTypePair("_version", std::make_shared<DataTypeUInt64>());
return IStorage::getColumn(column_name);
}
bool StorageLiveView::hasColumn(const String & column_name) const
{
if (column_name == "_version")
return true;
return IStorage::hasColumn(column_name);
}
Block StorageLiveView::getHeader() const
{
if (!sample_block)
{
auto storage = global_context.getTable(select_database_name, select_table_name);
sample_block = InterpreterSelectQuery(inner_query, global_context, storage,
SelectQueryOptions(QueryProcessingStage::Complete)).getSampleBlock();
sample_block.insert({DataTypeUInt64().createColumnConst(
sample_block.rows(), 0)->convertToFullColumnIfConst(),
std::make_shared<DataTypeUInt64>(),
"_version"});
/// convert all columns to full columns
/// in case some of them are constant
for (size_t i = 0; i < sample_block.columns(); ++i)
{
sample_block.safeGetByPosition(i).column = sample_block.safeGetByPosition(i).column->convertToFullColumnIfConst();
}
}
return sample_block;
}
bool StorageLiveView::getNewBlocks()
{
SipHash hash;
UInt128 key;
BlocksPtr new_blocks = std::make_shared<Blocks>();
BlocksMetadataPtr new_blocks_metadata = std::make_shared<BlocksMetadata>();
BlocksPtr new_mergeable_blocks = std::make_shared<Blocks>();
InterpreterSelectQuery interpreter(inner_query->clone(), global_context, SelectQueryOptions(QueryProcessingStage::WithMergeableState), Names());
auto mergeable_stream = std::make_shared<MaterializingBlockInputStream>(interpreter.execute().in);
while (Block block = mergeable_stream->read())
new_mergeable_blocks->push_back(block);
mergeable_blocks = std::make_shared<std::vector<BlocksPtr>>();
mergeable_blocks->push_back(new_mergeable_blocks);
BlockInputStreamPtr from = std::make_shared<BlocksBlockInputStream>(std::make_shared<BlocksPtr>(new_mergeable_blocks), mergeable_stream->getHeader());
auto proxy_storage = ProxyStorage::createProxyStorage(global_context.getTable(select_database_name, select_table_name), {from}, QueryProcessingStage::WithMergeableState);
InterpreterSelectQuery select(inner_query->clone(), global_context, proxy_storage, SelectQueryOptions(QueryProcessingStage::Complete));
BlockInputStreamPtr data = std::make_shared<MaterializingBlockInputStream>(select.execute().in);
/// Squashing is needed here because the view query can generate a lot of blocks
/// even when only one block is inserted into the parent table (e.g. if the query is a GROUP BY
/// and two-level aggregation is triggered).
data = std::make_shared<SquashingBlockInputStream>(
data, global_context.getSettingsRef().min_insert_block_size_rows, global_context.getSettingsRef().min_insert_block_size_bytes);
while (Block block = data->read())
{
/// calculate hash before virtual column is added
block.updateHash(hash);
/// add result version meta column
block.insert({DataTypeUInt64().createColumnConst(
block.rows(), getBlocksVersion() + 1)->convertToFullColumnIfConst(),
std::make_shared<DataTypeUInt64>(),
"_version"});
new_blocks->push_back(block);
}
hash.get128(key.low, key.high);
/// Update blocks only if hash keys do not match
/// NOTE: hash could be different for the same result
/// if blocks are not in the same order
bool updated = false;
{
if (getBlocksHashKey() != key.toHexString())
{
if (new_blocks->empty())
{
new_blocks->push_back(getHeader());
}
new_blocks_metadata->hash = key.toHexString();
new_blocks_metadata->version = getBlocksVersion() + 1;
(*blocks_ptr) = new_blocks;
(*blocks_metadata_ptr) = new_blocks_metadata;
updated = true;
}
}
return updated;
}
void StorageLiveView::checkTableCanBeDropped() const
{
Dependencies dependencies = global_context.getDependencies(database_name, table_name);
if (!dependencies.empty())
{
DatabaseAndTableName database_and_table_name = dependencies.front();
throw Exception("Table has dependency " + database_and_table_name.first + "." + database_and_table_name.second, ErrorCodes::TABLE_WAS_NOT_DROPPED);
}
}
void StorageLiveView::noUsersThread(const UInt64 & timeout)
{
if (shutdown_called)
return;
bool drop_table = false;
{
while (1)
{
Poco::FastMutex::ScopedLock lock(noUsersThreadMutex);
if (!noUsersThreadWakeUp && !noUsersThreadCondition.tryWait(noUsersThreadMutex,
timeout * 1000))
{
noUsersThreadWakeUp = false;
if (shutdown_called)
return;
if (hasUsers())
return;
if (!global_context.getDependencies(database_name, table_name).empty())
continue;
drop_table = true;
}
break;
}
}
if (drop_table)
{
if (global_context.tryGetTable(database_name, table_name))
{
try
{
/// We create and execute `drop` query for this table
auto drop_query = std::make_shared<ASTDropQuery>();
drop_query->database = database_name;
drop_query->table = table_name;
drop_query->kind = ASTDropQuery::Kind::Drop;
ASTPtr ast_drop_query = drop_query;
InterpreterDropQuery drop_interpreter(ast_drop_query, global_context);
drop_interpreter.execute();
}
catch (...)
{
}
}
}
}
void StorageLiveView::startNoUsersThread(const UInt64 & timeout)
{
bool expected = false;
if (!startnousersthread_called.compare_exchange_strong(expected, true))
return;
if (is_dropped)
return;
if (is_temporary)
{
if (no_users_thread.joinable())
{
{
Poco::FastMutex::ScopedLock lock(noUsersThreadMutex);
noUsersThreadWakeUp = true;
noUsersThreadCondition.signal();
}
no_users_thread.join();
}
{
Poco::FastMutex::ScopedLock lock(noUsersThreadMutex);
noUsersThreadWakeUp = false;
}
if (!is_dropped)
no_users_thread = std::thread(&StorageLiveView::noUsersThread, this, timeout);
}
startnousersthread_called = false;
}
void StorageLiveView::startup()
{
startNoUsersThread(temporary_live_view_timeout);
}
void StorageLiveView::shutdown()
{
bool expected = false;
if (!shutdown_called.compare_exchange_strong(expected, true))
return;
if (no_users_thread.joinable())
{
Poco::FastMutex::ScopedLock lock(noUsersThreadMutex);
noUsersThreadWakeUp = true;
noUsersThreadCondition.signal();
/// Must detach the no users thread
/// as we can't join it as it will result
/// in a deadlock
no_users_thread.detach();
}
}
StorageLiveView::~StorageLiveView()
{
shutdown();
}
void StorageLiveView::drop()
{
global_context.removeDependency(
DatabaseAndTableName(select_database_name, select_table_name),
DatabaseAndTableName(database_name, table_name));
Poco::FastMutex::ScopedLock lock(mutex);
is_dropped = true;
condition.broadcast();
}
void StorageLiveView::refresh(const Context & context)
{
auto alter_lock = lockAlterIntention(context.getCurrentQueryId());
{
Poco::FastMutex::ScopedLock lock(mutex);
if (getNewBlocks())
condition.broadcast();
}
}
BlockInputStreams StorageLiveView::read(
const Names & /*column_names*/,
const SelectQueryInfo & /*query_info*/,
const Context & /*context*/,
QueryProcessingStage::Enum /*processed_stage*/,
const size_t /*max_block_size*/,
const unsigned /*num_streams*/)
{
/// add user to the blocks_ptr
std::shared_ptr<BlocksPtr> stream_blocks_ptr = blocks_ptr;
{
Poco::FastMutex::ScopedLock lock(mutex);
if (!(*blocks_ptr))
{
if (getNewBlocks())
condition.broadcast();
}
}
return { std::make_shared<BlocksBlockInputStream>(stream_blocks_ptr, getHeader()) };
}
BlockInputStreams StorageLiveView::watch(
const Names & /*column_names*/,
const SelectQueryInfo & query_info,
const Context & context,
QueryProcessingStage::Enum & processed_stage,
size_t /*max_block_size*/,
const unsigned /*num_streams*/)
{
ASTWatchQuery & query = typeid_cast<ASTWatchQuery &>(*query_info.query);
bool has_limit = false;
UInt64 limit = 0;
if (query.limit_length)
{
has_limit = true;
limit = safeGet<UInt64>(typeid_cast<ASTLiteral &>(*query.limit_length).value);
}
if (query.is_watch_events)
{
auto reader = std::make_shared<LiveViewEventsBlockInputStream>(std::static_pointer_cast<StorageLiveView>(shared_from_this()), blocks_ptr, blocks_metadata_ptr, active_ptr, has_limit, limit, context.getSettingsRef().live_view_heartbeat_interval.totalSeconds(),
context.getSettingsRef().temporary_live_view_timeout.totalSeconds());
if (no_users_thread.joinable())
{
Poco::FastMutex::ScopedLock lock(noUsersThreadMutex);
noUsersThreadWakeUp = true;
noUsersThreadCondition.signal();
}
{
Poco::FastMutex::ScopedLock lock(mutex);
if (!(*blocks_ptr))
{
if (getNewBlocks())
condition.broadcast();
}
}
processed_stage = QueryProcessingStage::Complete;
return { reader };
}
else
{
auto reader = std::make_shared<LiveViewBlockInputStream>(std::static_pointer_cast<StorageLiveView>(shared_from_this()), blocks_ptr, blocks_metadata_ptr, active_ptr, has_limit, limit, context.getSettingsRef().live_view_heartbeat_interval.totalSeconds(),
context.getSettingsRef().temporary_live_view_timeout.totalSeconds());
if (no_users_thread.joinable())
{
Poco::FastMutex::ScopedLock lock(noUsersThreadMutex);
noUsersThreadWakeUp = true;
noUsersThreadCondition.signal();
}
{
Poco::FastMutex::ScopedLock lock(mutex);
if (!(*blocks_ptr))
{
if (getNewBlocks())
condition.broadcast();
}
}
processed_stage = QueryProcessingStage::Complete;
return { reader };
}
}
void registerStorageLiveView(StorageFactory & factory)
{
factory.registerStorage("LiveView", [](const StorageFactory::Arguments & args)
{
return StorageLiveView::create(args.table_name, args.database_name, args.local_context, args.query, args.columns);
});
}
}

View File

@ -0,0 +1,347 @@
/* Copyright (c) 2018 BlackBerry Limited
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
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. */
#pragma once
#include <Poco/Condition.h>
#include <Interpreters/InterpreterSelectQuery.h>
#include <DataStreams/IBlockOutputStream.h>
#include <DataStreams/OneBlockInputStream.h>
#include <DataStreams/BlocksBlockInputStream.h>
#include <DataStreams/MaterializingBlockInputStream.h>
#include <DataStreams/SquashingBlockInputStream.h>
#include <DataStreams/copyData.h>
#include <DataTypes/DataTypesNumber.h>
#include <ext/shared_ptr_helper.h>
#include <Common/SipHash.h>
#include <Storages/IStorage.h>
#include <Storages/ProxyStorage.h>
namespace DB
{
class IAST;
struct BlocksMetadata
{
String hash;
UInt64 version;
};
using ASTPtr = std::shared_ptr<IAST>;
using BlocksMetadataPtr = std::shared_ptr<BlocksMetadata>;
using SipHashPtr = std::shared_ptr<SipHash>;
class StorageLiveView : public ext::shared_ptr_helper<StorageLiveView>, public IStorage
{
friend struct ext::shared_ptr_helper<StorageLiveView>;
friend class LiveViewBlockOutputStream;
public:
~StorageLiveView() override;
String getName() const override { return "LiveView"; }
String getTableName() const override { return table_name; }
String getDatabaseName() const override { return database_name; }
String getSelectDatabaseName() const { return select_database_name; }
String getSelectTableName() const { return select_table_name; }
NameAndTypePair getColumn(const String & column_name) const override;
bool hasColumn(const String & column_name) const override;
// const NamesAndTypesList & getColumnsListImpl() const override { return *columns; }
ASTPtr getInnerQuery() const { return inner_query->clone(); }
/// It is passed inside the query and solved at its level.
bool supportsSampling() const override { return true; }
bool supportsFinal() const override { return true; }
/// Mutex for the blocks and ready condition
Poco::FastMutex mutex;
/// New blocks ready condition to broadcast to readers
/// that new blocks are available
Poco::Condition condition;
bool isTemporary() { return is_temporary; }
/// Check if we have any readers
/// must be called with mutex locked
bool hasUsers()
{
return blocks_ptr.use_count() > 1;
}
/// Check we have any active readers
/// must be called with mutex locked
bool hasActiveUsers()
{
return active_ptr.use_count() > 1;
}
/// Background thread for temporary tables
/// which drops this table if there are no users
void startNoUsersThread(const UInt64 & timeout);
Poco::FastMutex noUsersThreadMutex;
bool noUsersThreadWakeUp{false};
Poco::Condition noUsersThreadCondition;
/// Get blocks hash
/// must be called with mutex locked
String getBlocksHashKey()
{
if (*blocks_metadata_ptr)
return (*blocks_metadata_ptr)->hash;
return "";
}
/// Get blocks version
/// must be called with mutex locked
UInt64 getBlocksVersion()
{
if (*blocks_metadata_ptr)
return (*blocks_metadata_ptr)->version;
return 0;
}
/// Reset blocks
/// must be called with mutex locked
void reset()
{
(*blocks_ptr).reset();
if (*blocks_metadata_ptr)
(*blocks_metadata_ptr)->hash.clear();
mergeable_blocks.reset();
}
void checkTableCanBeDropped() const override;
void drop() override;
void startup() override;
void shutdown() override;
void refresh(const Context & context);
BlockInputStreams read(
const Names & column_names,
const SelectQueryInfo & query_info,
const Context & context,
QueryProcessingStage::Enum processed_stage,
size_t max_block_size,
unsigned num_streams) override;
BlockInputStreams watch(
const Names & column_names,
const SelectQueryInfo & query_info,
const Context & context,
QueryProcessingStage::Enum & processed_stage,
size_t max_block_size,
unsigned num_streams) override;
std::shared_ptr<BlocksPtr> getBlocksPtr() { return blocks_ptr; }
BlocksPtrs getMergeableBlocks() { return mergeable_blocks; }
void setMergeableBlocks(BlocksPtrs blocks) { mergeable_blocks = blocks; }
std::shared_ptr<bool> getActivePtr() { return active_ptr; }
/// Read new data blocks that store query result
bool getNewBlocks();
Block getHeader() const;
static void writeIntoLiveView(StorageLiveView & live_view,
const Block & block,
const Context & context,
BlockOutputStreamPtr & output)
{
/// Check if live view has any readers if not
/// just reset blocks to empty and do nothing else
/// When first reader comes the blocks will be read.
{
Poco::FastMutex::ScopedLock lock(live_view.mutex);
if (!live_view.hasActiveUsers())
{
live_view.reset();
return;
}
}
bool is_block_processed = false;
BlockInputStreams from;
BlocksPtrs mergeable_blocks;
BlocksPtr new_mergeable_blocks = std::make_shared<Blocks>();
{
Poco::FastMutex::ScopedLock lock(live_view.mutex);
mergeable_blocks = live_view.getMergeableBlocks();
if (!mergeable_blocks || mergeable_blocks->size() >= context.getGlobalContext().getSettingsRef().max_live_view_insert_blocks_before_refresh)
{
mergeable_blocks = std::make_shared<std::vector<BlocksPtr>>();
BlocksPtr base_mergeable_blocks = std::make_shared<Blocks>();
InterpreterSelectQuery interpreter(live_view.getInnerQuery(), context, SelectQueryOptions(QueryProcessingStage::WithMergeableState), Names());
auto view_mergeable_stream = std::make_shared<MaterializingBlockInputStream>(
interpreter.execute().in);
while (Block this_block = view_mergeable_stream->read())
base_mergeable_blocks->push_back(this_block);
mergeable_blocks->push_back(base_mergeable_blocks);
live_view.setMergeableBlocks(mergeable_blocks);
/// Create from streams
for (auto & blocks_ : *mergeable_blocks)
{
if (blocks_->empty())
continue;
auto sample_block = blocks_->front().cloneEmpty();
BlockInputStreamPtr stream = std::make_shared<BlocksBlockInputStream>(std::make_shared<BlocksPtr>(blocks_), sample_block);
from.push_back(std::move(stream));
}
is_block_processed = true;
}
}
if (!is_block_processed)
{
auto parent_storage = context.getTable(live_view.getSelectDatabaseName(), live_view.getSelectTableName());
BlockInputStreams streams = {std::make_shared<OneBlockInputStream>(block)};
auto proxy_storage = std::make_shared<ProxyStorage>(parent_storage, std::move(streams), QueryProcessingStage::FetchColumns);
InterpreterSelectQuery select_block(live_view.getInnerQuery(),
context, proxy_storage,
QueryProcessingStage::WithMergeableState);
auto data_mergeable_stream = std::make_shared<MaterializingBlockInputStream>(
select_block.execute().in);
while (Block this_block = data_mergeable_stream->read())
new_mergeable_blocks->push_back(this_block);
if (new_mergeable_blocks->empty())
return;
{
Poco::FastMutex::ScopedLock lock(live_view.mutex);
mergeable_blocks = live_view.getMergeableBlocks();
mergeable_blocks->push_back(new_mergeable_blocks);
/// Create from streams
for (auto & blocks_ : *mergeable_blocks)
{
if (blocks_->empty())
continue;
auto sample_block = blocks_->front().cloneEmpty();
BlockInputStreamPtr stream = std::make_shared<BlocksBlockInputStream>(std::make_shared<BlocksPtr>(blocks_), sample_block);
from.push_back(std::move(stream));
}
}
}
auto parent_storage = context.getTable(live_view.getSelectDatabaseName(), live_view.getSelectTableName());
auto proxy_storage = std::make_shared<ProxyStorage>(parent_storage, std::move(from), QueryProcessingStage::WithMergeableState);
InterpreterSelectQuery select(live_view.getInnerQuery(), context, proxy_storage, QueryProcessingStage::Complete);
BlockInputStreamPtr data = std::make_shared<MaterializingBlockInputStream>(select.execute().in);
/// Squashing is needed here because the view query can generate a lot of blocks
/// even when only one block is inserted into the parent table (e.g. if the query is a GROUP BY
/// and two-level aggregation is triggered).
data = std::make_shared<SquashingBlockInputStream>(
data, context.getGlobalContext().getSettingsRef().min_insert_block_size_rows, context.getGlobalContext().getSettingsRef().min_insert_block_size_bytes);
copyData(*data, *output);
}
private:
String select_database_name;
String select_table_name;
String table_name;
String database_name;
ASTPtr inner_query;
Context & global_context;
bool is_temporary {false};
mutable Block sample_block;
/// Active users
std::shared_ptr<bool> active_ptr;
/// Current data blocks that store query result
std::shared_ptr<BlocksPtr> blocks_ptr;
/// Current data blocks metadata
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr;
BlocksPtrs mergeable_blocks;
void noUsersThread(const UInt64 & timeout);
std::thread no_users_thread;
std::atomic<bool> shutdown_called{false};
std::atomic<bool> startnousersthread_called{false};
UInt64 temporary_live_view_timeout;
StorageLiveView(
const String & table_name_,
const String & database_name_,
Context & local_context,
const ASTCreateQuery & query,
const ColumnsDescription & columns
);
};
class LiveViewBlockOutputStream : public IBlockOutputStream
{
public:
explicit LiveViewBlockOutputStream(StorageLiveView & storage_) : storage(storage_) {}
void writePrefix() override
{
new_blocks = std::make_shared<Blocks>();
new_blocks_metadata = std::make_shared<BlocksMetadata>();
new_hash = std::make_shared<SipHash>();
}
void writeSuffix() override
{
UInt128 key;
String key_str;
new_hash->get128(key.low, key.high);
key_str = key.toHexString();
Poco::FastMutex::ScopedLock lock(storage.mutex);
if (storage.getBlocksHashKey() != key_str)
{
new_blocks_metadata->hash = key_str;
new_blocks_metadata->version = storage.getBlocksVersion() + 1;
for (auto & block : *new_blocks)
{
block.insert({DataTypeUInt64().createColumnConst(
block.rows(), new_blocks_metadata->version)->convertToFullColumnIfConst(),
std::make_shared<DataTypeUInt64>(),
"_version"});
}
(*storage.blocks_ptr) = new_blocks;
(*storage.blocks_metadata_ptr) = new_blocks_metadata;
storage.condition.broadcast();
}
new_blocks.reset();
new_blocks_metadata.reset();
new_hash.reset();
}
void write(const Block & block) override
{
new_blocks->push_back(block);
block.updateHash(*new_hash);
}
Block getHeader() const override { return storage.getHeader(); }
private:
BlocksPtr new_blocks;
BlocksMetadataPtr new_blocks_metadata;
SipHashPtr new_hash;
StorageLiveView & storage;
};
}

View File

@ -206,29 +206,35 @@ std::vector<MergeTreeData::AlterDataPartTransactionPtr> StorageMergeTree::prepar
const Settings & settings_ = context.getSettingsRef();
size_t thread_pool_size = std::min<size_t>(parts.size(), settings_.max_alter_threads);
ThreadPool thread_pool(thread_pool_size);
std::optional<ThreadPool> thread_pool;
if (thread_pool_size > 1)
thread_pool.emplace(thread_pool_size);
for (const auto & part : parts)
{
transactions.push_back(std::make_unique<MergeTreeData::AlterDataPartTransaction>(part));
thread_pool.schedule(
[this, & transaction = transactions.back(), & columns_for_parts, & new_indices = new_indices.indices]
{
this->alterDataPart(columns_for_parts, new_indices, false, transaction);
}
);
auto job = [this, & transaction = transactions.back(), & columns_for_parts, & new_indices = new_indices.indices]
{
this->alterDataPart(columns_for_parts, new_indices, false, transaction);
};
if (thread_pool)
thread_pool->schedule(job);
else
job();
}
thread_pool.wait();
if (thread_pool)
thread_pool->wait();
auto erase_pos = std::remove_if(transactions.begin(), transactions.end(),
[](const MergeTreeData::AlterDataPartTransactionPtr & transaction)
{
return !transaction->isValid();
}
);
});
transactions.erase(erase_pos, transactions.end());
return transactions;
@ -596,7 +602,7 @@ bool StorageMergeTree::merge(
bool force_ttl = (final && (hasTableTTL() || hasAnyColumnTTL()));
new_part = merger_mutator.mergePartsToTemporaryPart(
future_part, *merge_entry, time(nullptr),
future_part, *merge_entry, table_lock_holder, time(nullptr),
merging_tagger->reserved_space.get(), deduplicate, force_ttl);
merger_mutator.renameMergedTemporaryPart(new_part, future_part.parts, nullptr);
removeEmptyColumnsFromPart(new_part);
@ -714,7 +720,7 @@ bool StorageMergeTree::tryMutatePart()
try
{
new_part = merger_mutator.mutatePartToTemporaryPart(future_part, commands, *merge_entry, global_context);
new_part = merger_mutator.mutatePartToTemporaryPart(future_part, commands, *merge_entry, global_context, table_lock_holder);
renameTempPartAndReplace(new_part);
tagger->is_successful = true;
write_part_log({});

View File

@ -91,7 +91,6 @@ namespace ErrorCodes
extern const int NOT_FOUND_NODE;
extern const int NO_ACTIVE_REPLICAS;
extern const int LEADERSHIP_CHANGED;
extern const int TABLE_IS_READ_ONLY;
extern const int TABLE_WAS_NOT_DROPPED;
extern const int PARTITION_ALREADY_EXISTS;
extern const int TOO_MANY_RETRIES_TO_FETCH_PARTS;
@ -1088,7 +1087,7 @@ bool StorageReplicatedMergeTree::tryExecuteMerge(const LogEntry & entry)
try
{
part = merger_mutator.mergePartsToTemporaryPart(
future_merged_part, *merge_entry, entry.create_time, reserved_space.get(), entry.deduplicate, entry.force_ttl);
future_merged_part, *merge_entry, table_lock, entry.create_time, reserved_space.get(), entry.deduplicate, entry.force_ttl);
merger_mutator.renameMergedTemporaryPart(part, parts, &transaction);
removeEmptyColumnsFromPart(part);
@ -1217,7 +1216,7 @@ bool StorageReplicatedMergeTree::tryExecutePartMutation(const StorageReplicatedM
try
{
new_part = merger_mutator.mutatePartToTemporaryPart(future_mutated_part, commands, *merge_entry, global_context);
new_part = merger_mutator.mutatePartToTemporaryPart(future_mutated_part, commands, *merge_entry, global_context, table_lock);
renameTempPartAndReplace(new_part, nullptr, &transaction);
try
@ -3111,9 +3110,10 @@ bool StorageReplicatedMergeTree::optimize(const ASTPtr & query, const ASTPtr & p
}
}
/// TODO: Bad setting name for such purpose
if (query_context.getSettingsRef().replication_alter_partitions_sync != 0)
{
/// NOTE Table lock must not be held while waiting. Some combination of R-W-R locks from different threads will yield to deadlock.
/// TODO Check all other "wait" places.
for (auto & merge_entry : merge_entries)
waitForAllReplicasToProcessLogEntry(merge_entry);
}

View File

@ -470,10 +470,15 @@ private:
/** Wait until all replicas, including this, execute the specified action from the log.
* If replicas are added at the same time, it can not wait the added replica .
*
* NOTE: This method must be called without table lock held.
* Because it effectively waits for other thread that usually has to also acquire a lock to proceed and this yields deadlock.
* TODO: There are wrong usages of this method that are not fixed yet.
*/
void waitForAllReplicasToProcessLogEntry(const ReplicatedMergeTreeLogEntryData & entry);
/** Wait until the specified replica executes the specified action from the log.
* NOTE: See comment about locks above.
*/
void waitForReplicaToProcessLogEntry(const String & replica_name, const ReplicatedMergeTreeLogEntryData & entry);

View File

@ -5,9 +5,13 @@
#include <DataStreams/LimitBlockInputStream.h>
#include <Storages/System/StorageSystemNumbers.h>
namespace DB
{
namespace
{
class NumbersBlockInputStream : public IBlockInputStream
{
public:
@ -43,8 +47,64 @@ private:
};
StorageSystemNumbers::StorageSystemNumbers(const std::string & name_, bool multithreaded_, std::optional<UInt64> limit_, UInt64 offset_)
: name(name_), multithreaded(multithreaded_), limit(limit_), offset(offset_)
struct NumbersMultiThreadedState
{
std::atomic<UInt64> counter;
explicit NumbersMultiThreadedState(UInt64 offset) : counter(offset) {}
};
using NumbersMultiThreadedStatePtr = std::shared_ptr<NumbersMultiThreadedState>;
class NumbersMultiThreadedBlockInputStream : public IBlockInputStream
{
public:
NumbersMultiThreadedBlockInputStream(NumbersMultiThreadedStatePtr state_, UInt64 block_size_, UInt64 max_counter_)
: state(std::move(state_)), block_size(block_size_), max_counter(max_counter_) {}
String getName() const override { return "NumbersMt"; }
Block getHeader() const override
{
return { ColumnWithTypeAndName(ColumnUInt64::create(), std::make_shared<DataTypeUInt64>(), "number") };
}
protected:
Block readImpl() override
{
if (block_size == 0)
return {};
UInt64 curr = state->counter.fetch_add(block_size, std::memory_order_acquire);
if (curr >= max_counter)
return {};
if (curr + block_size > max_counter)
block_size = max_counter - curr;
auto column = ColumnUInt64::create(block_size);
ColumnUInt64::Container & vec = column->getData();
UInt64 * pos = vec.data();
UInt64 * end = &vec[block_size];
while (pos < end)
*pos++ = curr++;
return { ColumnWithTypeAndName(std::move(column), std::make_shared<DataTypeUInt64>(), "number") };
}
private:
NumbersMultiThreadedStatePtr state;
UInt64 block_size;
UInt64 max_counter;
};
}
StorageSystemNumbers::StorageSystemNumbers(const std::string & name_, bool multithreaded_, std::optional<UInt64> limit_, UInt64 offset_, bool even_distribution_)
: name(name_), multithreaded(multithreaded_), even_distribution(even_distribution_), limit(limit_), offset(offset_)
{
setColumns(ColumnsDescription({{"number", std::make_shared<DataTypeUInt64>()}}));
}
@ -69,6 +129,18 @@ BlockInputStreams StorageSystemNumbers::read(
num_streams = 1;
BlockInputStreams res(num_streams);
if (num_streams > 1 && !even_distribution && *limit)
{
auto state = std::make_shared<NumbersMultiThreadedState>(offset);
UInt64 max_counter = offset + *limit;
for (size_t i = 0; i < num_streams; ++i)
res[i] = std::make_shared<NumbersMultiThreadedBlockInputStream>(state, max_block_size, max_counter);
return res;
}
for (size_t i = 0; i < num_streams; ++i)
{
res[i] = std::make_shared<NumbersBlockInputStream>(max_block_size, offset + i * max_block_size, num_streams * max_block_size);

View File

@ -19,6 +19,9 @@ class Context;
* If multithreaded is specified, numbers will be generated in several streams
* (and result could be out of order). If both multithreaded and limit are specified,
* the table could give you not exactly 1..limit range, but some arbitrary 'limit' numbers.
*
* In multithreaded case, if even_distributed is False, implementation with atomic is used,
* and result is always in [0 ... limit - 1] range.
*/
class StorageSystemNumbers : public ext::shared_ptr_helper<StorageSystemNumbers>, public IStorage
{
@ -38,11 +41,14 @@ public:
private:
const std::string name;
bool multithreaded;
bool even_distribution;
std::optional<UInt64> limit;
UInt64 offset;
protected:
StorageSystemNumbers(const std::string & name_, bool multithreaded_, std::optional<UInt64> limit_ = std::nullopt, UInt64 offset_ = 0);
/// If even_distribution is true, numbers are distributed evenly between streams.
/// Otherwise, streams concurrently increment atomic.
StorageSystemNumbers(const std::string & name_, bool multithreaded_, std::optional<UInt64> limit_ = std::nullopt, UInt64 offset_ = 0, bool even_distribution_ = true);
};
}

View File

@ -24,6 +24,8 @@ void registerStorageSet(StorageFactory & factory);
void registerStorageJoin(StorageFactory & factory);
void registerStorageView(StorageFactory & factory);
void registerStorageMaterializedView(StorageFactory & factory);
void registerStorageLiveView(StorageFactory & factory);
//void registerStorageLiveChannel(StorageFactory & factory);
#if USE_HDFS
void registerStorageHDFS(StorageFactory & factory);
@ -64,6 +66,8 @@ void registerStorages()
registerStorageJoin(factory);
registerStorageView(factory);
registerStorageMaterializedView(factory);
registerStorageLiveView(factory);
//registerStorageLiveChannel(factory);
#if USE_HDFS
registerStorageHDFS(factory);

View File

@ -16,34 +16,35 @@ namespace ErrorCodes
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
}
StoragePtr TableFunctionNumbers::executeImpl(const ASTPtr & ast_function, const Context & context, const std::string & table_name) const
template <bool multithreaded>
StoragePtr TableFunctionNumbers<multithreaded>::executeImpl(const ASTPtr & ast_function, const Context & context, const std::string & table_name) const
{
if (const auto * function = ast_function->as<ASTFunction>())
{
auto arguments = function->arguments->children;
if (arguments.size() != 1 && arguments.size() != 2)
throw Exception("Table function 'numbers' requires 'length' or 'offset, length'.", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
throw Exception("Table function '" + getName() + "' requires 'length' or 'offset, length'.", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
UInt64 offset = arguments.size() == 2 ? evaluateArgument(context, arguments[0]) : 0;
UInt64 length = arguments.size() == 2 ? evaluateArgument(context, arguments[1]) : evaluateArgument(context, arguments[0]);
auto res = StorageSystemNumbers::create(table_name, false, length, offset);
auto res = StorageSystemNumbers::create(table_name, multithreaded, length, offset, false);
res->startup();
return res;
}
throw Exception("Table function 'numbers' requires 'limit' or 'offset, limit'.", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
throw Exception("Table function '" + getName() + "' requires 'limit' or 'offset, limit'.", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
}
void registerTableFunctionNumbers(TableFunctionFactory & factory)
{
factory.registerFunction<TableFunctionNumbers>();
factory.registerFunction<TableFunctionNumbers<true>>();
factory.registerFunction<TableFunctionNumbers<false>>();
}
UInt64 TableFunctionNumbers::evaluateArgument(const Context & context, ASTPtr & argument) const
template <bool multithreaded>
UInt64 TableFunctionNumbers<multithreaded>::evaluateArgument(const Context & context, ASTPtr & argument) const
{
return evaluateConstantExpressionOrIdentifierAsLiteral(argument, context)->as<ASTLiteral &>().value.safeGet<UInt64>();
}

View File

@ -7,14 +7,15 @@
namespace DB
{
/* numbers(limit)
/* numbers(limit), numbers_mt(limit)
* - the same as SELECT number FROM system.numbers LIMIT limit.
* Used for testing purposes, as a simple example of table function.
*/
template <bool multithreaded>
class TableFunctionNumbers : public ITableFunction
{
public:
static constexpr auto name = "numbers";
static constexpr auto name = multithreaded ? "numbers_mt" : "numbers";
std::string getName() const override { return name; }
private:
StoragePtr executeImpl(const ASTPtr & ast_function, const Context & context, const std::string & table_name) const override;

View File

@ -40,7 +40,7 @@ ninja
## Start ClickHouse and run tests
```
sudo -u clickhouse TSAN_OPTIONS='halt_on_error=1,suppressions=../dbms/tests/tsan_suppressions.txt' ./clickhouse-tsan server --config /etc/clickhouse-server/config.xml
sudo -u clickhouse TSAN_OPTIONS='halt_on_error=1' ./clickhouse-tsan server --config /etc/clickhouse-server/config.xml
```

View File

@ -1,18 +1,19 @@
<test>
<type>once</type>
<type>loop</type>
<stop_conditions>
<all_of>
<total_time_ms>10000</total_time_ms>
<all_of>
<iterations>3</iterations>
<min_time_not_changing_for_ms>10000</min_time_not_changing_for_ms>
</all_of>
<any_of>
<average_speed_not_changing_for_ms>5000</average_speed_not_changing_for_ms>
<total_time_ms>20000</total_time_ms>
<iterations>5</iterations>
<total_time_ms>60000</total_time_ms>
</any_of>
</stop_conditions>
<main_metric>
<max_bytes_per_second/>
<min_time/>
</main_metric>
<substitutions>
@ -45,11 +46,11 @@
<substitution>
<name>table</name>
<values>
<value>numbers</value>
<value>numbers_mt</value>
<value>numbers(1000000)</value>
<value>numbers_mt(10000000)</value>
</values>
</substitution>
</substitutions>
<query>SELECT ignore({gp_hash_func}({string})) FROM system.{table}</query>
<query>SELECT count() from {table} where not ignore({gp_hash_func}({string}))</query>
</test>

View File

@ -2,5 +2,5 @@ SELECT groupArrayInsertAt(toString(number), number * 2) FROM (SELECT * FROM syst
SELECT groupArrayInsertAt('-')(toString(number), number * 2) FROM (SELECT * FROM system.numbers LIMIT 10);
SELECT groupArrayInsertAt([123])(range(number), number * 2) FROM (SELECT * FROM system.numbers LIMIT 10);
SELECT number, groupArrayInsertAt(number, number) FROM (SELECT * FROM system.numbers LIMIT 10) GROUP BY number ORDER BY number;
SELECT k, ignore(groupArrayInsertAt(x, x)) FROM (SELECT dummy AS k, randConstant() % 10 AS x FROM remote('127.0.0.{1,1}', system.one)) GROUP BY k ORDER BY k;
SELECT k, ignore(groupArrayInsertAt(x, x)) FROM (SELECT dummy AS k, (randConstant() * 10) % 10 AS x FROM remote('127.0.0.{1,1}', system.one)) GROUP BY k ORDER BY k;
SELECT k, groupArrayInsertAt('-', 10)(toString(x), x) FROM (SELECT number AS k, number AS x FROM system.numbers LIMIT 11) GROUP BY k ORDER BY k;

View File

@ -20,7 +20,7 @@ INSERT INTO memory SELECT * FROM numbers(1000);"
${CLICKHOUSE_CLIENT} --multiquery --query="
SET max_threads = 1;
SELECT count() FROM memory WHERE NOT ignore(sleep(0.0001));" 2>&1 | grep -c -P '^1000$|Table .+? doesn.t exist' &
SELECT count() FROM memory WHERE NOT ignore(sleep(0.0001));" 2>&1 | grep -c -P '^1000$|^0$|Table .+? doesn.t exist' &
sleep 0.05;

File diff suppressed because one or more lines are too long

View File

@ -13,7 +13,7 @@ $CLICKHOUSE_CLIENT -q "select name from system.table_functions format TSV;" > $S
# if you want long run use: env SQL_FUZZY_RUNS=100000 clickhouse-test sql_fuzzy
for SQL_FUZZY_RUN in $(seq ${SQL_FUZZY_RUNS:=10}); do
env SQL_FUZZY_RUN=$SQL_FUZZY_RUN $CURDIR/00746_sql_fuzzy.pl | $CLICKHOUSE_CLIENT -n --ignore-error >/dev/null 2>&1
env SQL_FUZZY_RUN=$SQL_FUZZY_RUN $CURDIR/00746_sql_fuzzy.pl | $CLICKHOUSE_CLIENT --max_execution_time 10 -n --ignore-error >/dev/null 2>&1
if [[ `$CLICKHOUSE_CLIENT -q "SELECT 'Still alive'"` != 'Still alive' ]]; then
break
fi

View File

@ -1,5 +1,5 @@
a 2018-01-01 00:00:00 0000-00-00 00:00:00
b 2018-01-01 00:00:00 b 2018-01-01 00:00:00
c 2018-01-01 00:00:00 c 2018-01-01 00:00:00
b 2018-01-01 00:00:00 b 2018-01-01 00:00:00
c 2018-01-01 00:00:00 c 2018-01-01 00:00:00
a 2018-01-01 00:00:00 0000-00-00 00:00:00
b 2018-01-01 00:00:00 b b 2018-01-01 00:00:00
c 2018-01-01 00:00:00 c c 2018-01-01 00:00:00
b 2018-01-01 00:00:00 b b 2018-01-01 00:00:00
c 2018-01-01 00:00:00 c c 2018-01-01 00:00:00

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