2019-05-26 22:03:30 +00:00
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#include <DataStreams/IBlockInputStream.h>
|
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2019-07-09 12:41:55 +00:00
|
|
|
/** Implements LIVE VIEW table WATCH input stream.
|
|
|
|
* Keeps stream alive by outputing blocks with no rows
|
|
|
|
* based on period specified by the heartbeat interval.
|
|
|
|
*/
|
2019-05-26 22:03:30 +00:00
|
|
|
class LiveViewBlockInputStream : public IBlockInputStream
|
|
|
|
{
|
|
|
|
|
|
|
|
using NonBlockingResult = std::pair<Block, bool>;
|
|
|
|
|
|
|
|
public:
|
|
|
|
~LiveViewBlockInputStream() override
|
|
|
|
{
|
2019-08-25 01:40:24 +00:00
|
|
|
/// Wake up no users thread
|
|
|
|
{
|
|
|
|
std::cerr << "DEBUG: live view block input stream ... send wake up thread\n";
|
|
|
|
std::lock_guard lock(storage->no_users_thread_mutex);
|
|
|
|
storage->no_users_thread_wakeup = true;
|
|
|
|
storage->no_users_thread_condition.notify_one();
|
|
|
|
}
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
2019-07-09 12:41:55 +00:00
|
|
|
|
2019-06-12 13:11:44 +00:00
|
|
|
LiveViewBlockInputStream(std::shared_ptr<StorageLiveView> storage_,
|
|
|
|
std::shared_ptr<BlocksPtr> blocks_ptr_,
|
2019-05-28 21:17:48 +00:00
|
|
|
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr_,
|
2019-06-12 13:11:44 +00:00
|
|
|
std::shared_ptr<bool> active_ptr_,
|
2019-07-09 12:41:55 +00:00
|
|
|
const bool has_limit_, const UInt64 limit_,
|
2019-08-25 01:40:24 +00:00
|
|
|
const UInt64 heartbeat_interval_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)
|
2019-05-26 22:03:30 +00:00
|
|
|
{
|
|
|
|
/// grab active pointer
|
|
|
|
active = active_ptr.lock();
|
|
|
|
}
|
|
|
|
|
|
|
|
String getName() const override { return "LiveViewBlockInputStream"; }
|
|
|
|
|
|
|
|
void cancel(bool kill) override
|
|
|
|
{
|
2019-06-12 13:11:44 +00:00
|
|
|
if (isCancelled() || storage->is_dropped)
|
2019-05-26 22:03:30 +00:00
|
|
|
return;
|
|
|
|
IBlockInputStream::cancel(kill);
|
2019-08-22 23:22:57 +00:00
|
|
|
std::lock_guard lock(storage->mutex);
|
|
|
|
storage->condition.notify_all();
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
2019-06-12 13:11:44 +00:00
|
|
|
Block getHeader() const override { return storage->getHeader(); }
|
2019-05-26 22:03:30 +00:00
|
|
|
|
|
|
|
void refresh()
|
|
|
|
{
|
|
|
|
if (active && blocks && it == end)
|
2019-07-09 12:41:55 +00:00
|
|
|
it = blocks->begin();
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
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()
|
|
|
|
{
|
2019-08-22 23:22:57 +00:00
|
|
|
return tryReadImpl(false);
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
protected:
|
|
|
|
Block readImpl() override
|
|
|
|
{
|
|
|
|
/// try reading
|
2019-08-22 23:22:57 +00:00
|
|
|
return tryReadImpl(true).first;
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
/** 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.
|
|
|
|
*/
|
2019-08-22 23:22:57 +00:00
|
|
|
NonBlockingResult tryReadImpl(bool blocking)
|
2019-05-26 22:03:30 +00:00
|
|
|
{
|
|
|
|
Block res;
|
|
|
|
|
2019-07-10 02:06:29 +00:00
|
|
|
if (has_limit && num_updates == static_cast<Int64>(limit))
|
2019-05-26 22:03:30 +00:00
|
|
|
{
|
|
|
|
return { Block(), true };
|
|
|
|
}
|
|
|
|
/// If blocks were never assigned get blocks
|
|
|
|
if (!blocks)
|
|
|
|
{
|
2019-08-22 23:22:57 +00:00
|
|
|
std::lock_guard lock(storage->mutex);
|
2019-05-26 22:03:30 +00:00
|
|
|
if (!active)
|
|
|
|
return { Block(), false };
|
|
|
|
blocks = (*blocks_ptr);
|
|
|
|
it = blocks->begin();
|
|
|
|
begin = blocks->begin();
|
|
|
|
end = blocks->end();
|
|
|
|
}
|
|
|
|
|
2019-06-12 13:11:44 +00:00
|
|
|
if (isCancelled() || storage->is_dropped)
|
2019-05-26 22:03:30 +00:00
|
|
|
{
|
|
|
|
return { Block(), true };
|
|
|
|
}
|
|
|
|
|
|
|
|
if (it == end)
|
|
|
|
{
|
|
|
|
{
|
2019-08-22 23:22:57 +00:00
|
|
|
std::unique_lock lock(storage->mutex);
|
2019-05-26 22:03:30 +00:00
|
|
|
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 };
|
|
|
|
}
|
2019-06-05 11:30:29 +00:00
|
|
|
if (!end_of_blocks)
|
|
|
|
{
|
|
|
|
end_of_blocks = true;
|
|
|
|
return { getHeader(), true };
|
|
|
|
}
|
2019-05-26 22:03:30 +00:00
|
|
|
while (true)
|
|
|
|
{
|
2019-06-11 12:27:47 +00:00
|
|
|
UInt64 timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
|
2019-08-22 23:22:57 +00:00
|
|
|
|
|
|
|
/// Or spurious wakeup.
|
|
|
|
bool signaled = std::cv_status::no_timeout == storage->condition.wait_for(lock,
|
|
|
|
std::chrono::microseconds(std::max(UInt64(0), heartbeat_interval_usec - (timestamp_usec - last_event_timestamp_usec))));
|
2019-05-26 22:03:30 +00:00
|
|
|
|
2019-06-12 13:11:44 +00:00
|
|
|
if (isCancelled() || storage->is_dropped)
|
2019-05-26 22:03:30 +00:00
|
|
|
{
|
|
|
|
return { Block(), true };
|
|
|
|
}
|
|
|
|
if (signaled)
|
|
|
|
{
|
|
|
|
break;
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2019-06-05 11:30:29 +00:00
|
|
|
// heartbeat
|
2019-07-09 12:41:55 +00:00
|
|
|
last_event_timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
|
2019-06-05 11:30:29 +00:00
|
|
|
return { getHeader(), true };
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2019-08-22 23:22:57 +00:00
|
|
|
return tryReadImpl(blocking);
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
res = *it;
|
|
|
|
|
|
|
|
++it;
|
|
|
|
|
|
|
|
if (it == end)
|
|
|
|
{
|
2019-06-05 11:30:29 +00:00
|
|
|
end_of_blocks = false;
|
2019-07-09 12:41:55 +00:00
|
|
|
num_updates += 1;
|
2019-05-26 22:03:30 +00:00
|
|
|
}
|
|
|
|
|
2019-07-09 12:41:55 +00:00
|
|
|
last_event_timestamp_usec = static_cast<UInt64>(timestamp.epochMicroseconds());
|
2019-05-26 22:03:30 +00:00
|
|
|
return { res, true };
|
|
|
|
}
|
|
|
|
|
|
|
|
private:
|
2019-06-12 13:11:44 +00:00
|
|
|
std::shared_ptr<StorageLiveView> storage;
|
2019-05-26 22:03:30 +00:00
|
|
|
std::shared_ptr<BlocksPtr> blocks_ptr;
|
2019-05-28 21:17:48 +00:00
|
|
|
std::shared_ptr<BlocksMetadataPtr> blocks_metadata_ptr;
|
2019-05-26 22:03:30 +00:00
|
|
|
std::weak_ptr<bool> active_ptr;
|
|
|
|
std::shared_ptr<bool> active;
|
|
|
|
BlocksPtr blocks;
|
2019-05-28 21:17:48 +00:00
|
|
|
BlocksMetadataPtr blocks_metadata;
|
2019-05-26 22:03:30 +00:00
|
|
|
Blocks::iterator it;
|
|
|
|
Blocks::iterator end;
|
|
|
|
Blocks::iterator begin;
|
2019-07-09 12:41:55 +00:00
|
|
|
const bool has_limit;
|
|
|
|
const UInt64 limit;
|
|
|
|
Int64 num_updates = -1;
|
|
|
|
bool end_of_blocks = false;
|
|
|
|
UInt64 heartbeat_interval_usec;
|
|
|
|
UInt64 last_event_timestamp_usec = 0;
|
2019-05-26 22:03:30 +00:00
|
|
|
Poco::Timestamp timestamp;
|
|
|
|
};
|
|
|
|
|
|
|
|
}
|