2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/StorageJoin.h>
|
2017-12-30 00:36:06 +00:00
|
|
|
#include <Storages/StorageFactory.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/Join.h>
|
2017-12-30 00:36:06 +00:00
|
|
|
#include <Parsers/ASTIdentifier.h>
|
|
|
|
#include <Common/typeid_cast.h>
|
|
|
|
|
2018-03-11 00:15:26 +00:00
|
|
|
#include <Poco/String.h> /// toLower
|
2018-06-09 16:09:37 +00:00
|
|
|
#include <Poco/File.h>
|
2015-01-27 21:24:24 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2016-01-11 21:46:36 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const int NO_SUCH_COLUMN_IN_TABLE;
|
|
|
|
extern const int INCOMPATIBLE_TYPE_OF_JOIN;
|
2017-12-30 00:36:06 +00:00
|
|
|
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
|
|
|
|
extern const int BAD_ARGUMENTS;
|
2016-01-11 21:46:36 +00:00
|
|
|
}
|
|
|
|
|
2015-01-27 21:24:24 +00:00
|
|
|
|
|
|
|
StorageJoin::StorageJoin(
|
2017-04-01 07:20:54 +00:00
|
|
|
const String & path_,
|
|
|
|
const String & name_,
|
|
|
|
const Names & key_names_,
|
|
|
|
ASTTableJoin::Kind kind_, ASTTableJoin::Strictness strictness_,
|
2018-03-06 20:18:34 +00:00
|
|
|
const ColumnsDescription & columns_)
|
|
|
|
: StorageSetOrJoinBase{path_, name_, columns_},
|
2017-04-01 07:20:54 +00:00
|
|
|
key_names(key_names_), kind(kind_), strictness(strictness_)
|
2015-01-27 21:24:24 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
for (const auto & key : key_names)
|
2018-03-13 15:00:28 +00:00
|
|
|
if (!getColumns().hasPhysical(key))
|
2018-05-07 02:01:11 +00:00
|
|
|
throw Exception{"Key column (" + key + ") does not exist in table declaration.", ErrorCodes::NO_SUCH_COLUMN_IN_TABLE};
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-04 06:52:39 +00:00
|
|
|
/// NOTE StorageJoin doesn't use join_use_nulls setting.
|
|
|
|
|
2018-03-11 00:15:26 +00:00
|
|
|
join = std::make_shared<Join>(key_names, key_names, false /* use_nulls */, SizeLimits(), kind, strictness);
|
Fixes StorageJoin's sample block order.
Here is a reproducible test case.
```
create table e (s UInt64, t UInt64) Engine = Memory;
create table v (s UInt64, w Float64, c UInt64) Engine = Join(Any, Inner, s);
insert into e values (1, 2), (1, 3), (1, 4), (2, 1), (2, 4), (3, 1), (4, 2), (4, 3);
insert into v values (1, 0.5, 3), (2, 0.5, 2), (3, 1, 1), (4, 0.5, 2);
select *, w, c from e any inner join v using (s);
```
# before this patch
```
┌─s─┬─t─┬─────w─┬──────────c─┐
│ 1 │ 2 │ 4e-45 │ 1051372192 │
│ 1 │ 3 │ 4e-45 │ 1051372192 │
│ 1 │ 4 │ 4e-45 │ 1051372192 │
│ 2 │ 1 │ 3e-45 │ 1056964608 │
│ 2 │ 4 │ 3e-45 │ 1056964608 │
│ 3 │ 1 │ 1e-45 │ 1065353216 │
│ 4 │ 2 │ 3e-45 │ 1056964608 │
│ 4 │ 3 │ 3e-45 │ 1056964608 │
└───┴───┴───────┴────────────┘
```
# after this patch
```
┌─s─┬─t─┬───w─┬─c─┐
│ 1 │ 2 │ 0.5 │ 3 │
│ 1 │ 3 │ 0.5 │ 3 │
│ 1 │ 4 │ 0.5 │ 3 │
│ 2 │ 1 │ 0.5 │ 2 │
│ 2 │ 4 │ 0.5 │ 2 │
│ 3 │ 1 │ 1 │ 1 │
│ 4 │ 2 │ 0.5 │ 2 │
│ 4 │ 3 │ 0.5 │ 2 │
└───┴───┴─────┴───┘
```
2017-11-20 11:46:24 +00:00
|
|
|
join->setSampleBlock(getSampleBlock().sortColumns());
|
2017-04-01 07:20:54 +00:00
|
|
|
restore();
|
2015-01-27 21:24:24 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2018-06-09 16:09:37 +00:00
|
|
|
void StorageJoin::truncate(const ASTPtr &)
|
|
|
|
{
|
|
|
|
Poco::File(path).remove(true);
|
|
|
|
Poco::File(path).createDirectories();
|
2018-06-09 18:17:27 +00:00
|
|
|
Poco::File(path + "tmp/").createDirectories();
|
2018-06-09 16:09:37 +00:00
|
|
|
|
|
|
|
increment = 0;
|
|
|
|
join = std::make_shared<Join>(key_names, key_names, false /* use_nulls */, SizeLimits(), kind, strictness);
|
|
|
|
join->setSampleBlock(getSampleBlock().sortColumns());
|
|
|
|
};
|
|
|
|
|
|
|
|
|
2016-07-22 20:39:28 +00:00
|
|
|
void StorageJoin::assertCompatible(ASTTableJoin::Kind kind_, ASTTableJoin::Strictness strictness_) const
|
2015-01-28 02:37:05 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
/// NOTE Could be more loose.
|
|
|
|
if (!(kind == kind_ && strictness == strictness_))
|
2018-03-06 20:18:34 +00:00
|
|
|
throw Exception("Table " + table_name + " has incompatible type of JOIN.", ErrorCodes::INCOMPATIBLE_TYPE_OF_JOIN);
|
2015-01-28 02:37:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-01-14 09:00:19 +00:00
|
|
|
void StorageJoin::insertBlock(const Block & block) { join->insertFromBlock(block); }
|
|
|
|
size_t StorageJoin::getSize() const { return join->getTotalRowCount(); };
|
|
|
|
|
2017-12-30 00:36:06 +00:00
|
|
|
|
|
|
|
void registerStorageJoin(StorageFactory & factory)
|
|
|
|
{
|
|
|
|
factory.registerStorage("Join", [](const StorageFactory::Arguments & args)
|
|
|
|
{
|
|
|
|
/// Join(ANY, LEFT, k1, k2, ...)
|
|
|
|
|
|
|
|
ASTs & engine_args = args.engine_args;
|
|
|
|
|
|
|
|
if (engine_args.size() < 3)
|
|
|
|
throw Exception(
|
|
|
|
"Storage Join requires at least 3 parameters: Join(ANY|ALL, LEFT|INNER, keys...).",
|
|
|
|
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
|
|
|
|
|
|
|
|
const ASTIdentifier * strictness_id = typeid_cast<const ASTIdentifier *>(engine_args[0].get());
|
|
|
|
if (!strictness_id)
|
|
|
|
throw Exception("First parameter of storage Join must be ANY or ALL (without quotes).", ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
|
|
const String strictness_str = Poco::toLower(strictness_id->name);
|
|
|
|
ASTTableJoin::Strictness strictness;
|
|
|
|
if (strictness_str == "any")
|
|
|
|
strictness = ASTTableJoin::Strictness::Any;
|
|
|
|
else if (strictness_str == "all")
|
|
|
|
strictness = ASTTableJoin::Strictness::All;
|
|
|
|
else
|
|
|
|
throw Exception("First parameter of storage Join must be ANY or ALL (without quotes).", ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
|
|
const ASTIdentifier * kind_id = typeid_cast<const ASTIdentifier *>(engine_args[1].get());
|
|
|
|
if (!kind_id)
|
|
|
|
throw Exception("Second parameter of storage Join must be LEFT or INNER (without quotes).", ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
|
|
const String kind_str = Poco::toLower(kind_id->name);
|
|
|
|
ASTTableJoin::Kind kind;
|
|
|
|
if (kind_str == "left")
|
|
|
|
kind = ASTTableJoin::Kind::Left;
|
|
|
|
else if (kind_str == "inner")
|
|
|
|
kind = ASTTableJoin::Kind::Inner;
|
|
|
|
else if (kind_str == "right")
|
|
|
|
kind = ASTTableJoin::Kind::Right;
|
|
|
|
else if (kind_str == "full")
|
|
|
|
kind = ASTTableJoin::Kind::Full;
|
|
|
|
else
|
|
|
|
throw Exception("Second parameter of storage Join must be LEFT or INNER or RIGHT or FULL (without quotes).", ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
|
|
Names key_names;
|
|
|
|
key_names.reserve(engine_args.size() - 2);
|
|
|
|
for (size_t i = 2, size = engine_args.size(); i < size; ++i)
|
|
|
|
{
|
|
|
|
const ASTIdentifier * key = typeid_cast<const ASTIdentifier *>(engine_args[i].get());
|
|
|
|
if (!key)
|
|
|
|
throw Exception("Parameter №" + toString(i + 1) + " of storage Join don't look like column name.", ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
|
|
key_names.push_back(key->name);
|
|
|
|
}
|
|
|
|
|
|
|
|
return StorageJoin::create(
|
|
|
|
args.data_path, args.table_name,
|
|
|
|
key_names, kind, strictness,
|
2018-03-06 20:18:34 +00:00
|
|
|
args.columns);
|
2017-12-30 00:36:06 +00:00
|
|
|
});
|
|
|
|
}
|
|
|
|
|
2015-01-27 21:24:24 +00:00
|
|
|
}
|