2013-02-07 13:03:19 +00:00
|
|
|
#include <DB/Storages/StorageChunks.h>
|
2013-06-17 07:01:31 +00:00
|
|
|
#include <DB/Storages/StorageChunkRef.h>
|
2013-02-07 13:03:19 +00:00
|
|
|
#include <DB/Common/escapeForFileName.h>
|
|
|
|
#include <DB/IO/ReadHelpers.h>
|
|
|
|
#include <DB/IO/WriteHelpers.h>
|
|
|
|
#include <DB/Interpreters/InterpreterDropQuery.h>
|
|
|
|
#include <DB/Parsers/ASTDropQuery.h>
|
2014-01-17 15:19:20 +00:00
|
|
|
#include <DB/Common/VirtualColumnUtils.h>
|
2013-02-07 13:03:19 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2013-05-05 18:02:05 +00:00
|
|
|
StoragePtr StorageChunks::create(
|
|
|
|
const std::string & path_,
|
|
|
|
const std::string & name_,
|
|
|
|
const std::string & database_name_,
|
|
|
|
NamesAndTypesListPtr columns_,
|
|
|
|
Context & context_,
|
|
|
|
bool attach)
|
2013-02-07 13:03:19 +00:00
|
|
|
{
|
2013-02-08 17:06:29 +00:00
|
|
|
return (new StorageChunks(path_, name_, database_name_, columns_, context_, attach))->thisPtr();
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void StorageChunks::addReference()
|
|
|
|
{
|
2013-02-08 17:06:29 +00:00
|
|
|
reference_counter.add(1, false);
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void StorageChunks::removeReference()
|
|
|
|
{
|
|
|
|
Int64 c = reference_counter.add(-1, false);
|
|
|
|
if (c < 0)
|
2013-02-07 13:40:59 +00:00
|
|
|
throw Exception("Negative refcount on table " + name, ErrorCodes::NEGATIVE_REFCOUNT);
|
2013-02-07 13:03:19 +00:00
|
|
|
if (c == 0)
|
|
|
|
dropThis();
|
|
|
|
}
|
|
|
|
|
2014-02-11 18:38:21 +00:00
|
|
|
BlockInputStreams StorageChunks::read(
|
|
|
|
const Names & column_names,
|
|
|
|
ASTPtr query,
|
|
|
|
const Settings & settings,
|
|
|
|
QueryProcessingStage::Enum & processed_stage,
|
|
|
|
size_t max_block_size,
|
|
|
|
unsigned threads)
|
|
|
|
{
|
2014-02-26 10:59:56 +00:00
|
|
|
bool has_virtual_column = false;
|
|
|
|
|
|
|
|
for (const auto & column : column_names)
|
|
|
|
if (column == _table_column_name)
|
|
|
|
has_virtual_column = true;
|
|
|
|
|
|
|
|
/// Если виртуальных столбцов нет, просто считать данные из таблицы
|
|
|
|
if (!has_virtual_column)
|
|
|
|
return read(0, std::numeric_limits<size_t>::max(), column_names, query, settings, processed_stage, max_block_size, threads);
|
|
|
|
|
2014-02-11 18:38:21 +00:00
|
|
|
Block virtual_columns_block = getBlockWithVirtualColumns();
|
|
|
|
BlockInputStreamPtr virtual_columns =
|
|
|
|
VirtualColumnUtils::getVirtualColumnsBlocks(query->clone(), virtual_columns_block, context);
|
2014-03-19 12:35:27 +00:00
|
|
|
std::multiset<String> values = VirtualColumnUtils::extractSingleValueFromBlocks<String>(virtual_columns, _table_column_name);
|
2014-02-11 18:38:21 +00:00
|
|
|
bool all_inclusive = (values.size() == virtual_columns_block.rows());
|
|
|
|
|
|
|
|
if (all_inclusive)
|
|
|
|
return read(0, std::numeric_limits<size_t>::max(), column_names, query, settings, processed_stage, max_block_size, threads);
|
|
|
|
|
|
|
|
BlockInputStreams res;
|
|
|
|
for (const auto & it : values)
|
|
|
|
{
|
|
|
|
BlockInputStreams temp = readFromChunk(it, column_names, query, settings, processed_stage, max_block_size, threads);
|
|
|
|
res.insert(res.end(), temp.begin(), temp.end());
|
|
|
|
}
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Построить блок состоящий только из возможных значений виртуальных столбцов
|
|
|
|
Block StorageChunks::getBlockWithVirtualColumns() const
|
|
|
|
{
|
|
|
|
Block res;
|
|
|
|
ColumnWithNameAndType _table(new ColumnString, new DataTypeString, _table_column_name);
|
|
|
|
|
|
|
|
for (const auto & it : chunk_names)
|
|
|
|
_table.column->insert(it);
|
|
|
|
|
|
|
|
res.insert(_table);
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
2013-02-07 13:03:19 +00:00
|
|
|
BlockInputStreams StorageChunks::readFromChunk(
|
|
|
|
const std::string & chunk_name,
|
|
|
|
const Names & column_names,
|
|
|
|
ASTPtr query,
|
|
|
|
const Settings & settings,
|
|
|
|
QueryProcessingStage::Enum & processed_stage,
|
|
|
|
size_t max_block_size,
|
|
|
|
unsigned threads)
|
|
|
|
{
|
|
|
|
size_t mark1;
|
|
|
|
size_t mark2;
|
|
|
|
|
|
|
|
{
|
|
|
|
Poco::ScopedReadRWLock lock(rwlock);
|
|
|
|
|
|
|
|
if (!chunk_indices.count(chunk_name))
|
2013-02-07 13:40:59 +00:00
|
|
|
throw Exception("No chunk " + chunk_name + " in table " + name, ErrorCodes::CHUNK_NOT_FOUND);
|
2013-02-07 13:03:19 +00:00
|
|
|
size_t index = chunk_indices[chunk_name];
|
2013-06-17 07:01:31 +00:00
|
|
|
mark1 = chunk_num_to_marks[index];
|
|
|
|
mark2 = index + 1 == chunk_num_to_marks.size() ? marksCount() : chunk_num_to_marks[index + 1];
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
2014-01-16 14:52:13 +00:00
|
|
|
|
2014-01-17 15:19:20 +00:00
|
|
|
return read(mark1, mark2, column_names, query, settings, processed_stage, max_block_size, threads);
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
BlockOutputStreamPtr StorageChunks::writeToNewChunk(
|
|
|
|
const std::string & chunk_name)
|
|
|
|
{
|
|
|
|
{
|
|
|
|
Poco::ScopedWriteRWLock lock(rwlock);
|
2013-06-17 07:01:31 +00:00
|
|
|
|
2013-02-07 13:03:19 +00:00
|
|
|
if (chunk_indices.count(chunk_name))
|
2013-02-07 13:40:59 +00:00
|
|
|
throw Exception("Duplicate chunk name in table " + name, ErrorCodes::DUPLICATE_CHUNK_NAME);
|
2013-06-17 07:01:31 +00:00
|
|
|
|
2013-02-07 13:03:19 +00:00
|
|
|
size_t mark = marksCount();
|
2013-06-17 07:01:31 +00:00
|
|
|
chunk_indices[chunk_name] = chunk_num_to_marks.size();
|
2013-02-07 13:03:19 +00:00
|
|
|
appendChunkToIndex(chunk_name, mark);
|
2013-06-17 07:01:31 +00:00
|
|
|
chunk_num_to_marks.push_back(mark);
|
2014-01-17 15:19:20 +00:00
|
|
|
chunk_names.push_back(chunk_name);
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
|
2013-02-07 13:07:13 +00:00
|
|
|
return StorageLog::write(NULL);
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
|
2013-05-05 18:02:05 +00:00
|
|
|
StorageChunks::StorageChunks(
|
|
|
|
const std::string & path_,
|
|
|
|
const std::string & name_,
|
|
|
|
const std::string & database_name_,
|
|
|
|
NamesAndTypesListPtr columns_,
|
|
|
|
Context & context_,
|
|
|
|
bool attach)
|
|
|
|
:
|
2014-03-28 14:36:24 +00:00
|
|
|
StorageLog(path_, name_, columns_, context_.getSettings().max_compress_block_size),
|
2013-05-05 18:02:05 +00:00
|
|
|
database_name(database_name_),
|
|
|
|
reference_counter(path_ + escapeForFileName(name_) + "/refcount.txt"),
|
|
|
|
context(context_),
|
|
|
|
log(&Logger::get("StorageChunks"))
|
2013-02-08 17:06:29 +00:00
|
|
|
{
|
|
|
|
if (!attach)
|
|
|
|
reference_counter.add(1, true);
|
2013-06-17 07:01:31 +00:00
|
|
|
|
|
|
|
loadIndex();
|
|
|
|
|
|
|
|
/// Создадим все таблицы типа ChunkRef. Они должны располагаться в той же БД.
|
2013-06-17 07:57:58 +00:00
|
|
|
{
|
|
|
|
Poco::ScopedLock<Poco::Mutex> lock(context.getMutex());
|
|
|
|
for (ChunkIndices::const_iterator it = chunk_indices.begin(); it != chunk_indices.end(); ++it)
|
|
|
|
{
|
|
|
|
if (context.isTableExist(database_name, it->first))
|
|
|
|
{
|
2013-10-22 19:11:33 +00:00
|
|
|
LOG_WARNING(log, "Chunk " << it->first << " exists in more than one Chunks tables.");
|
2013-06-17 07:57:58 +00:00
|
|
|
context.detachTable(database_name, it->first);
|
|
|
|
}
|
|
|
|
|
|
|
|
context.addTable(database_name, it->first, StorageChunkRef::create(it->first, context, database_name, name, true));
|
|
|
|
}
|
|
|
|
}
|
2014-01-16 14:52:13 +00:00
|
|
|
|
2014-01-16 15:25:54 +00:00
|
|
|
_table_column_name = "_table" + VirtualColumnUtils::chooseSuffix(getColumnsList(), "_table");
|
2014-01-16 14:52:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
NameAndTypePair StorageChunks::getColumn(const String &column_name) const
|
|
|
|
{
|
|
|
|
if (column_name == _table_column_name) return std::make_pair(_table_column_name, new DataTypeString);
|
|
|
|
return getRealColumn(column_name);
|
|
|
|
}
|
|
|
|
|
|
|
|
bool StorageChunks::hasColumn(const String &column_name) const
|
|
|
|
{
|
|
|
|
if (column_name == _table_column_name) return true;
|
|
|
|
return hasRealColumn(column_name);
|
2013-02-08 17:06:29 +00:00
|
|
|
}
|
2014-01-17 15:19:20 +00:00
|
|
|
|
|
|
|
std::pair<String, size_t> StorageChunks::getTableFromMark(size_t mark) const
|
|
|
|
{
|
|
|
|
/// Находим последний <= элемент в массие
|
|
|
|
size_t pos = std::upper_bound(chunk_num_to_marks.begin(), chunk_num_to_marks.end(), mark) - chunk_num_to_marks.begin() - 1;
|
|
|
|
/// Вычисляем номер засечки до которой будет длится текущая таблица
|
|
|
|
size_t last = std::numeric_limits<size_t>::max();
|
|
|
|
if (pos + 1 < chunk_num_to_marks.size())
|
|
|
|
last = chunk_num_to_marks[pos + 1] - 1;
|
|
|
|
return std::make_pair(chunk_names[pos], last);
|
|
|
|
}
|
2013-02-07 13:03:19 +00:00
|
|
|
|
|
|
|
void StorageChunks::loadIndex()
|
|
|
|
{
|
|
|
|
loadMarks();
|
2013-06-17 07:01:31 +00:00
|
|
|
|
2013-02-07 13:03:19 +00:00
|
|
|
Poco::ScopedWriteRWLock lock(rwlock);
|
|
|
|
|
|
|
|
String index_path = path + escapeForFileName(name) + "/chunks.chn";
|
2013-02-11 10:11:59 +00:00
|
|
|
|
|
|
|
if (!Poco::File(index_path).exists())
|
|
|
|
return;
|
|
|
|
|
2013-02-07 13:03:19 +00:00
|
|
|
ReadBufferFromFile index(index_path, 4096);
|
|
|
|
while (!index.eof())
|
|
|
|
{
|
|
|
|
String name;
|
|
|
|
size_t mark;
|
|
|
|
|
|
|
|
readStringBinary(name, index);
|
2013-02-07 13:07:13 +00:00
|
|
|
readIntBinary<UInt64>(mark, index);
|
2013-02-07 13:03:19 +00:00
|
|
|
|
2013-06-17 07:01:31 +00:00
|
|
|
chunk_indices[name] = chunk_num_to_marks.size();
|
|
|
|
chunk_num_to_marks.push_back(mark);
|
2014-01-17 15:19:20 +00:00
|
|
|
chunk_names.push_back(name);
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2013-02-11 11:42:36 +00:00
|
|
|
void StorageChunks::appendChunkToIndex(const std::string & chunk_name, size_t mark)
|
2013-02-07 13:03:19 +00:00
|
|
|
{
|
|
|
|
String index_path = path + escapeForFileName(name) + "/chunks.chn";
|
|
|
|
WriteBufferFromFile index(index_path, 4096, O_APPEND | O_CREAT | O_WRONLY);
|
2013-02-11 11:42:36 +00:00
|
|
|
writeStringBinary(chunk_name, index);
|
2013-02-07 13:03:19 +00:00
|
|
|
writeIntBinary<UInt64>(mark, index);
|
2013-09-26 19:16:43 +00:00
|
|
|
index.next();
|
2013-02-07 13:03:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void StorageChunks::dropThis()
|
|
|
|
{
|
2013-02-11 12:28:43 +00:00
|
|
|
LOG_TRACE(log, "Table " << name << " will drop itself.");
|
|
|
|
|
2013-02-07 13:03:19 +00:00
|
|
|
ASTDropQuery * query = new ASTDropQuery();
|
|
|
|
ASTPtr query_ptr = query;
|
|
|
|
query->detach = false;
|
|
|
|
query->if_exists = false;
|
|
|
|
query->database = database_name;
|
|
|
|
query->table = name;
|
|
|
|
|
|
|
|
InterpreterDropQuery interpreter(query_ptr, context);
|
|
|
|
interpreter.execute();
|
|
|
|
}
|
2013-02-07 13:07:13 +00:00
|
|
|
|
|
|
|
}
|