2020-04-28 13:55:50 +00:00
|
|
|
#include <Core/SortCursor.h>
|
|
|
|
#include <Interpreters/SortedBlocksWriter.h>
|
|
|
|
#include <DataStreams/MergingSortedBlockInputStream.h>
|
|
|
|
#include <DataStreams/OneBlockInputStream.h>
|
|
|
|
#include <DataStreams/TemporaryFileStream.h>
|
2021-01-26 12:37:42 +00:00
|
|
|
#include <DataStreams/materializeBlock.h>
|
2021-01-23 15:20:15 +00:00
|
|
|
#include <Disks/IVolume.h>
|
2020-04-28 13:55:50 +00:00
|
|
|
|
2021-07-17 18:06:46 +00:00
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int NOT_ENOUGH_SPACE;
|
|
|
|
}
|
|
|
|
|
|
|
|
namespace
|
|
|
|
{
|
|
|
|
|
|
|
|
std::unique_ptr<TemporaryFile> flushToFile(const String & tmp_path, const Block & header, IBlockInputStream & stream, const String & codec)
|
|
|
|
{
|
|
|
|
auto tmp_file = createTemporaryFile(tmp_path);
|
|
|
|
|
|
|
|
std::atomic<bool> is_cancelled{false};
|
|
|
|
TemporaryFileStream::write(tmp_file->path(), header, stream, &is_cancelled, codec);
|
|
|
|
if (is_cancelled)
|
|
|
|
throw Exception("Cannot flush MergeJoin data on disk. No space at " + tmp_path, ErrorCodes::NOT_ENOUGH_SPACE);
|
|
|
|
|
|
|
|
return tmp_file;
|
|
|
|
}
|
|
|
|
|
|
|
|
SortedBlocksWriter::SortedFiles flushToManyFiles(const String & tmp_path, const Block & header, IBlockInputStream & stream,
|
|
|
|
const String & codec, std::function<void(const Block &)> callback = [](const Block &){})
|
|
|
|
{
|
|
|
|
std::vector<std::unique_ptr<TemporaryFile>> files;
|
|
|
|
|
|
|
|
while (Block block = stream.read())
|
|
|
|
{
|
|
|
|
if (!block.rows())
|
|
|
|
continue;
|
|
|
|
|
|
|
|
callback(block);
|
|
|
|
|
|
|
|
OneBlockInputStream block_stream(block);
|
|
|
|
auto tmp_file = flushToFile(tmp_path, header, block_stream, codec);
|
|
|
|
files.emplace_back(std::move(tmp_file));
|
|
|
|
}
|
|
|
|
|
|
|
|
return files;
|
|
|
|
}
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void SortedBlocksWriter::insert(Block && block)
|
|
|
|
{
|
|
|
|
bool can_insert_more = false;
|
|
|
|
bool has_data_to_flush = false;
|
|
|
|
|
|
|
|
BlocksList current_blocks;
|
|
|
|
size_t row_count = 0;
|
|
|
|
size_t bytes = 0;
|
|
|
|
size_t flush_no = 0;
|
|
|
|
|
|
|
|
{
|
|
|
|
std::lock_guard lock{insert_mutex};
|
|
|
|
|
|
|
|
/// insert bock into BlocksList undef lock
|
|
|
|
inserted_blocks.insert(std::move(block));
|
|
|
|
|
|
|
|
size_t total_row_count = inserted_blocks.row_count + row_count_in_flush;
|
|
|
|
size_t total_bytes = inserted_blocks.bytes + bytes_in_flush;
|
|
|
|
|
|
|
|
can_insert_more = size_limits.softCheck(total_row_count, total_bytes);
|
|
|
|
has_data_to_flush = !size_limits.softCheck(inserted_blocks.row_count * num_streams, inserted_blocks.bytes * num_streams);
|
|
|
|
|
|
|
|
if (has_data_to_flush)
|
|
|
|
{
|
|
|
|
++flush_inflight;
|
|
|
|
current_blocks.swap(inserted_blocks.blocks);
|
|
|
|
row_count_in_flush = total_row_count;
|
|
|
|
bytes_in_flush = total_bytes;
|
|
|
|
|
|
|
|
row_count = inserted_blocks.row_count;
|
|
|
|
bytes = inserted_blocks.bytes;
|
|
|
|
inserted_blocks.clear();
|
|
|
|
}
|
|
|
|
else if (can_insert_more)
|
|
|
|
flush_no = flush_number;
|
|
|
|
}
|
|
|
|
|
|
|
|
if (has_data_to_flush)
|
|
|
|
{
|
|
|
|
/// flush new blocks without lock
|
|
|
|
auto flushed = flush(current_blocks);
|
|
|
|
current_blocks.clear();
|
|
|
|
|
|
|
|
std::lock_guard lock{insert_mutex};
|
|
|
|
|
|
|
|
sorted_files.emplace_back(std::move(flushed));
|
|
|
|
row_count_in_flush -= row_count;
|
|
|
|
bytes_in_flush -= bytes;
|
|
|
|
|
|
|
|
/// notify another insert (flush_number) and merge (flush_inflight)
|
|
|
|
++flush_number;
|
|
|
|
--flush_inflight;
|
|
|
|
flush_condvar.notify_all();
|
|
|
|
}
|
|
|
|
else if (!can_insert_more)
|
|
|
|
{
|
|
|
|
/// wakeup insert blocked by out of limit
|
|
|
|
std::unique_lock lock{insert_mutex};
|
|
|
|
flush_condvar.wait(lock, [&]{ return flush_no < flush_number; });
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
SortedBlocksWriter::TmpFilePtr SortedBlocksWriter::flush(const BlocksList & blocks) const
|
|
|
|
{
|
|
|
|
const std::string path = getPath();
|
|
|
|
|
|
|
|
if (blocks.empty())
|
|
|
|
return {};
|
|
|
|
|
|
|
|
if (blocks.size() == 1)
|
|
|
|
{
|
|
|
|
OneBlockInputStream sorted_input(blocks.front());
|
|
|
|
return flushToFile(path, sample_block, sorted_input, codec);
|
|
|
|
}
|
|
|
|
|
|
|
|
BlockInputStreams inputs;
|
|
|
|
inputs.reserve(blocks.size());
|
|
|
|
for (const auto & block : blocks)
|
|
|
|
if (block.rows())
|
|
|
|
inputs.push_back(std::make_shared<OneBlockInputStream>(block));
|
|
|
|
|
|
|
|
MergingSortedBlockInputStream sorted_input(inputs, sort_description, rows_in_block);
|
|
|
|
return flushToFile(path, sample_block, sorted_input, codec);
|
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
SortedBlocksWriter::PremergedFiles SortedBlocksWriter::premerge()
|
2020-04-28 13:55:50 +00:00
|
|
|
{
|
2020-06-16 20:13:18 +00:00
|
|
|
SortedFiles files;
|
|
|
|
BlocksList blocks;
|
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
/// wait other flushes if any
|
|
|
|
{
|
|
|
|
std::unique_lock lock{insert_mutex};
|
2020-06-16 20:13:18 +00:00
|
|
|
|
|
|
|
files.swap(sorted_files);
|
|
|
|
blocks.swap(inserted_blocks.blocks);
|
|
|
|
inserted_blocks.clear();
|
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
flush_condvar.wait(lock, [&]{ return !flush_inflight; });
|
|
|
|
}
|
|
|
|
|
|
|
|
/// flush not flushed
|
2020-06-16 20:13:18 +00:00
|
|
|
if (!blocks.empty())
|
|
|
|
files.emplace_back(flush(blocks));
|
2020-04-28 13:55:50 +00:00
|
|
|
|
|
|
|
BlockInputStreams inputs;
|
|
|
|
inputs.reserve(num_files_for_merge);
|
|
|
|
|
|
|
|
/// Merge by parts to save memory. It's possible to exchange disk I/O and memory by num_files_for_merge.
|
|
|
|
{
|
|
|
|
SortedFiles new_files;
|
2020-06-16 20:13:18 +00:00
|
|
|
new_files.reserve(files.size() / num_files_for_merge + 1);
|
2020-04-28 13:55:50 +00:00
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
while (files.size() > num_files_for_merge)
|
2020-04-28 13:55:50 +00:00
|
|
|
{
|
2020-06-16 20:13:18 +00:00
|
|
|
for (const auto & file : files)
|
2020-04-28 13:55:50 +00:00
|
|
|
{
|
|
|
|
inputs.emplace_back(streamFromFile(file));
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
if (inputs.size() == num_files_for_merge || &file == &files.back())
|
2020-04-28 13:55:50 +00:00
|
|
|
{
|
|
|
|
MergingSortedBlockInputStream sorted_input(inputs, sort_description, rows_in_block);
|
|
|
|
new_files.emplace_back(flushToFile(getPath(), sample_block, sorted_input, codec));
|
|
|
|
inputs.clear();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
files.clear();
|
|
|
|
files.swap(new_files);
|
2020-04-28 13:55:50 +00:00
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
for (const auto & file : files)
|
2020-04-28 13:55:50 +00:00
|
|
|
inputs.emplace_back(streamFromFile(file));
|
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
return PremergedFiles{std::move(files), std::move(inputs)};
|
|
|
|
}
|
2020-04-28 13:55:50 +00:00
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
SortedBlocksWriter::SortedFiles SortedBlocksWriter::finishMerge(std::function<void(const Block &)> callback)
|
|
|
|
{
|
|
|
|
PremergedFiles files = premerge();
|
|
|
|
MergingSortedBlockInputStream sorted_input(files.streams, sort_description, rows_in_block);
|
|
|
|
return flushToManyFiles(getPath(), sample_block, sorted_input, codec, callback);
|
2020-04-28 13:55:50 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
BlockInputStreamPtr SortedBlocksWriter::streamFromFile(const TmpFilePtr & file) const
|
|
|
|
{
|
2021-01-26 12:37:42 +00:00
|
|
|
return std::make_shared<TemporaryFileLazyInputStream>(file->path(), materializeBlock(sample_block));
|
2020-04-28 13:55:50 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
String SortedBlocksWriter::getPath() const
|
|
|
|
{
|
2020-05-28 05:38:55 +00:00
|
|
|
return volume->getDisk()->getPath();
|
2020-04-28 13:55:50 +00:00
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
|
|
|
|
Block SortedBlocksBuffer::exchange(Block && block)
|
|
|
|
{
|
|
|
|
static constexpr const float reserve_coef = 1.2;
|
|
|
|
|
|
|
|
Blocks out_blocks;
|
|
|
|
Block empty_out = block.cloneEmpty();
|
|
|
|
|
|
|
|
{
|
|
|
|
std::lock_guard lock(mutex);
|
|
|
|
|
|
|
|
if (block)
|
|
|
|
{
|
|
|
|
current_bytes += block.bytes();
|
|
|
|
buffer.emplace_back(std::move(block));
|
|
|
|
|
|
|
|
/// Saved. Return empty block with same structure.
|
|
|
|
if (current_bytes < max_bytes)
|
|
|
|
return empty_out;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Not saved. Return buffered.
|
|
|
|
out_blocks.swap(buffer);
|
|
|
|
buffer.reserve(out_blocks.size() * reserve_coef);
|
|
|
|
current_bytes = 0;
|
|
|
|
}
|
|
|
|
|
|
|
|
if (size_t size = out_blocks.size())
|
|
|
|
{
|
|
|
|
if (size == 1)
|
|
|
|
return out_blocks[0];
|
|
|
|
return mergeBlocks(std::move(out_blocks));
|
|
|
|
}
|
|
|
|
|
|
|
|
return {};
|
|
|
|
}
|
|
|
|
|
|
|
|
Block SortedBlocksBuffer::mergeBlocks(Blocks && blocks) const
|
|
|
|
{
|
|
|
|
size_t num_rows = 0;
|
|
|
|
|
|
|
|
{ /// Merge sort blocks
|
|
|
|
BlockInputStreams inputs;
|
|
|
|
inputs.reserve(blocks.size());
|
|
|
|
|
|
|
|
for (auto & block : blocks)
|
|
|
|
{
|
|
|
|
num_rows += block.rows();
|
|
|
|
inputs.emplace_back(std::make_shared<OneBlockInputStream>(block));
|
|
|
|
}
|
|
|
|
|
|
|
|
Blocks tmp_blocks;
|
|
|
|
MergingSortedBlockInputStream stream(inputs, sort_description, num_rows);
|
|
|
|
while (const auto & block = stream.read())
|
|
|
|
tmp_blocks.emplace_back(block);
|
|
|
|
|
|
|
|
blocks.swap(tmp_blocks);
|
|
|
|
}
|
|
|
|
|
|
|
|
if (blocks.size() == 1)
|
|
|
|
return blocks[0];
|
|
|
|
|
|
|
|
Block out = blocks[0].cloneEmpty();
|
|
|
|
|
|
|
|
{ /// Concatenate blocks
|
|
|
|
MutableColumns columns = out.mutateColumns();
|
|
|
|
|
|
|
|
for (size_t i = 0; i < columns.size(); ++i)
|
|
|
|
{
|
|
|
|
columns[i]->reserve(num_rows);
|
|
|
|
for (const auto & block : blocks)
|
|
|
|
{
|
|
|
|
const auto & tmp_column = *block.getByPosition(i).column;
|
|
|
|
columns[i]->insertRangeFrom(tmp_column, 0, block.rows());
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
out.setColumns(std::move(columns));
|
|
|
|
}
|
|
|
|
|
|
|
|
return out;
|
|
|
|
}
|
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
}
|