2019-05-26 22:03:30 +00:00
|
|
|
/* 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 <ext/shared_ptr_helper.h>
|
|
|
|
#include <Storages/IStorage.h>
|
2019-08-22 23:22:57 +00:00
|
|
|
|
|
|
|
#include <mutex>
|
|
|
|
#include <condition_variable>
|
2019-05-26 22:03:30 +00:00
|
|
|
|
2019-08-22 22:41:30 +00:00
|
|
|
|
2019-05-26 22:03:30 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2019-06-11 10:30:57 +00:00
|
|
|
struct BlocksMetadata
|
2019-06-10 11:41:33 +00:00
|
|
|
{
|
2019-05-28 21:17:48 +00:00
|
|
|
String hash;
|
|
|
|
UInt64 version;
|
|
|
|
};
|
|
|
|
|
2019-08-22 22:41:30 +00:00
|
|
|
class IAST;
|
2019-06-11 02:22:53 +00:00
|
|
|
using ASTPtr = std::shared_ptr<IAST>;
|
2019-05-28 21:17:48 +00:00
|
|
|
using BlocksMetadataPtr = std::shared_ptr<BlocksMetadata>;
|
|
|
|
|
2019-05-26 22:03:30 +00:00
|
|
|
class StorageLiveView : public ext::shared_ptr_helper<StorageLiveView>, public IStorage
|
|
|
|
{
|
|
|
|
friend struct ext::shared_ptr_helper<StorageLiveView>;
|
2019-08-22 23:22:57 +00:00
|
|
|
friend class LiveViewBlockInputStream;
|
|
|
|
friend class LiveViewEventsBlockInputStream;
|
2019-05-26 22:03:30 +00:00
|
|
|
friend class LiveViewBlockOutputStream;
|
|
|
|
|
|
|
|
public:
|
|
|
|
~StorageLiveView() override;
|
|
|
|
String getName() const override { return "LiveView"; }
|
|
|
|
String getTableName() const override { return table_name; }
|
2019-06-10 11:18:33 +00:00
|
|
|
String getDatabaseName() const override { return database_name; }
|
2019-05-26 22:03:30 +00:00
|
|
|
String getSelectDatabaseName() const { return select_database_name; }
|
|
|
|
String getSelectTableName() const { return select_table_name; }
|
|
|
|
|
2019-05-29 21:52:13 +00:00
|
|
|
NameAndTypePair getColumn(const String & column_name) const override;
|
|
|
|
bool hasColumn(const String & column_name) const override;
|
|
|
|
|
2019-05-26 22:03:30 +00:00
|
|
|
// const NamesAndTypesList & getColumnsListImpl() const override { return *columns; }
|
2019-06-10 11:18:33 +00:00
|
|
|
ASTPtr getInnerQuery() const { return inner_query->clone(); }
|
2019-12-04 23:29:37 +00:00
|
|
|
ASTPtr getInnerSubQuery() const
|
|
|
|
{
|
|
|
|
if (inner_subquery)
|
|
|
|
return inner_subquery->clone();
|
|
|
|
return nullptr;
|
|
|
|
};
|
|
|
|
ASTPtr getInnerOuterQuery() const
|
|
|
|
{
|
|
|
|
if (inner_outer_query)
|
|
|
|
return inner_outer_query->clone();
|
|
|
|
return nullptr;
|
|
|
|
};
|
2019-05-26 22:03:30 +00:00
|
|
|
|
|
|
|
/// It is passed inside the query and solved at its level.
|
|
|
|
bool supportsSampling() const override { return true; }
|
|
|
|
bool supportsFinal() const override { return true; }
|
|
|
|
|
|
|
|
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;
|
|
|
|
}
|
|
|
|
|
2019-07-09 12:41:55 +00:00
|
|
|
/// Check we have any active readers
|
2019-05-26 22:03:30 +00:00
|
|
|
/// must be called with mutex locked
|
|
|
|
bool hasActiveUsers()
|
|
|
|
{
|
|
|
|
return active_ptr.use_count() > 1;
|
|
|
|
}
|
2019-08-25 01:40:24 +00:00
|
|
|
/// No users thread mutex, predicate and wake up condition
|
2019-08-25 12:27:47 +00:00
|
|
|
void startNoUsersThread(const UInt64 & timeout);
|
2019-10-17 14:47:36 +00:00
|
|
|
std::mutex no_users_thread_wakeup_mutex;
|
2019-08-25 01:40:24 +00:00
|
|
|
bool no_users_thread_wakeup = false;
|
2019-08-22 23:22:57 +00:00
|
|
|
std::condition_variable no_users_thread_condition;
|
2019-05-30 21:29:30 +00:00
|
|
|
/// Get blocks hash
|
|
|
|
/// must be called with mutex locked
|
2019-05-26 22:03:30 +00:00
|
|
|
String getBlocksHashKey()
|
|
|
|
{
|
2019-05-28 21:17:48 +00:00
|
|
|
if (*blocks_metadata_ptr)
|
|
|
|
return (*blocks_metadata_ptr)->hash;
|
2019-08-22 23:22:57 +00:00
|
|
|
return {};
|
2019-05-28 21:17:48 +00:00
|
|
|
}
|
2019-05-30 21:29:30 +00:00
|
|
|
/// Get blocks version
|
|
|
|
/// must be called with mutex locked
|
2019-05-28 21:17:48 +00:00
|
|
|
UInt64 getBlocksVersion()
|
|
|
|
{
|
|
|
|
if (*blocks_metadata_ptr)
|
|
|
|
return (*blocks_metadata_ptr)->version;
|
2019-05-29 21:52:13 +00:00
|
|
|
return 0;
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
/// Reset blocks
|
|
|
|
/// must be called with mutex locked
|
|
|
|
void reset()
|
|
|
|
{
|
|
|
|
(*blocks_ptr).reset();
|
2019-06-07 01:10:56 +00:00
|
|
|
if (*blocks_metadata_ptr)
|
|
|
|
(*blocks_metadata_ptr)->hash.clear();
|
2019-05-26 22:03:30 +00:00
|
|
|
mergeable_blocks.reset();
|
|
|
|
}
|
|
|
|
|
|
|
|
void checkTableCanBeDropped() const override;
|
2019-08-27 20:43:08 +00:00
|
|
|
void drop(TableStructureWriteLockHolder &) override;
|
2019-05-26 22:03:30 +00:00
|
|
|
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; }
|
2019-06-17 16:39:01 +00:00
|
|
|
void setMergeableBlocks(BlocksPtrs blocks) { mergeable_blocks = blocks; }
|
2019-05-26 22:03:30 +00:00
|
|
|
std::shared_ptr<bool> getActivePtr() { return active_ptr; }
|
|
|
|
|
|
|
|
/// Read new data blocks that store query result
|
|
|
|
bool getNewBlocks();
|
|
|
|
|
|
|
|
Block getHeader() const;
|
|
|
|
|
2019-08-22 22:41:30 +00:00
|
|
|
static void writeIntoLiveView(
|
|
|
|
StorageLiveView & live_view,
|
|
|
|
const Block & block,
|
|
|
|
const Context & context);
|
2019-05-26 22:03:30 +00:00
|
|
|
|
|
|
|
private:
|
|
|
|
String select_database_name;
|
|
|
|
String select_table_name;
|
|
|
|
String table_name;
|
|
|
|
String database_name;
|
2019-12-04 23:29:37 +00:00
|
|
|
ASTPtr inner_query; /// stored query : SELECT * FROM ( SELECT a FROM A)
|
|
|
|
ASTPtr inner_subquery; /// stored query's subquery if any : SLECT a FROM A
|
|
|
|
ASTPtr inner_outer_query; /// the query right before innermost subquery : ... SELECT * FROM ( subquery )
|
2019-05-26 22:03:30 +00:00
|
|
|
Context & global_context;
|
2019-11-15 14:24:28 +00:00
|
|
|
std::unique_ptr<Context> live_view_context;
|
|
|
|
|
2019-08-25 01:40:24 +00:00
|
|
|
bool is_temporary = false;
|
2019-10-17 14:43:11 +00:00
|
|
|
/// Mutex to protect access to sample block
|
|
|
|
mutable std::mutex sample_block_lock;
|
2019-05-26 22:03:30 +00:00
|
|
|
mutable Block sample_block;
|
|
|
|
|
2019-08-22 23:22:57 +00:00
|
|
|
/// Mutex for the blocks and ready condition
|
|
|
|
std::mutex mutex;
|
|
|
|
/// New blocks ready condition to broadcast to readers
|
|
|
|
/// that new blocks are available
|
|
|
|
std::condition_variable condition;
|
|
|
|
|
2019-05-26 22:03:30 +00:00
|
|
|
/// Active users
|
|
|
|
std::shared_ptr<bool> active_ptr;
|
|
|
|
/// Current data blocks that store query result
|
|
|
|
std::shared_ptr<BlocksPtr> blocks_ptr;
|
2019-05-28 21:17:48 +00:00
|
|
|
/// Current data blocks metadata
|
|
|
|
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr;
|
2019-05-26 22:03:30 +00:00
|
|
|
BlocksPtrs mergeable_blocks;
|
|
|
|
|
2019-08-25 01:40:24 +00:00
|
|
|
/// Background thread for temporary tables
|
|
|
|
/// which drops this table if there are no users
|
|
|
|
static void noUsersThread(std::shared_ptr<StorageLiveView> storage, const UInt64 & timeout);
|
2019-10-17 14:47:36 +00:00
|
|
|
std::mutex no_users_thread_mutex;
|
2019-05-26 22:03:30 +00:00
|
|
|
std::thread no_users_thread;
|
2019-08-25 01:40:24 +00:00
|
|
|
std::atomic<bool> shutdown_called = false;
|
2019-08-25 12:27:47 +00:00
|
|
|
std::atomic<bool> start_no_users_thread_called = false;
|
2019-06-07 01:10:56 +00:00
|
|
|
UInt64 temporary_live_view_timeout;
|
2019-05-26 22:03:30 +00:00
|
|
|
|
|
|
|
StorageLiveView(
|
|
|
|
const String & table_name_,
|
|
|
|
const String & database_name_,
|
|
|
|
Context & local_context,
|
|
|
|
const ASTCreateQuery & query,
|
|
|
|
const ColumnsDescription & columns
|
|
|
|
);
|
|
|
|
};
|
|
|
|
|
|
|
|
}
|