mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-10 09:32:06 +00:00
fix some trash
This commit is contained in:
parent
712a04d69c
commit
dea39d8175
@ -15,6 +15,7 @@
|
||||
#include <Access/User.h>
|
||||
#include <Access/ExternalAuthenticators.h>
|
||||
#include <Core/Settings.h>
|
||||
#include <Core/Protocol.h>
|
||||
#include <base/find_symbols.h>
|
||||
#include <Poco/ExpireCache.h>
|
||||
#include <boost/algorithm/string/join.hpp>
|
||||
@ -164,6 +165,7 @@ void AccessControl::setUpFromMainConfig(const Poco::Util::AbstractConfiguration
|
||||
false /* false because we need to be compatible with earlier access configurations */));
|
||||
|
||||
addStoragesFromMainConfig(config_, config_path_, get_zookeeper_function_);
|
||||
checkNoReservedUsersInConfigs();
|
||||
}
|
||||
|
||||
|
||||
@ -179,6 +181,16 @@ void AccessControl::setUsersConfig(const Poco::Util::AbstractConfiguration & use
|
||||
}
|
||||
}
|
||||
addUsersConfigStorage(users_config_);
|
||||
checkNoReservedUsersInConfigs();
|
||||
}
|
||||
|
||||
void AccessControl::checkNoReservedUsersInConfigs()
|
||||
{
|
||||
/// Unfortunately, there is not way to distinguish USER_INTERSERVER_MARKER from actual username in native protocol,
|
||||
/// so we have to ensure that no such user will appear.
|
||||
/// Also it was possible to create a user with empty name for some reason.
|
||||
if (find<User>(USER_INTERSERVER_MARKER) || find<User>(""))
|
||||
throw Exception(ErrorCodes::UNKNOWN_ELEMENT_IN_CONFIG, "Found reserved username in configs");
|
||||
}
|
||||
|
||||
void AccessControl::addUsersConfigStorage(const Poco::Util::AbstractConfiguration & users_config_)
|
||||
@ -235,6 +247,7 @@ void AccessControl::reloadUsersConfigs()
|
||||
if (auto users_config_storage = typeid_cast<std::shared_ptr<UsersConfigAccessStorage>>(storage))
|
||||
users_config_storage->reload();
|
||||
}
|
||||
checkNoReservedUsersInConfigs();
|
||||
}
|
||||
|
||||
void AccessControl::startPeriodicReloadingUsersConfigs()
|
||||
|
@ -57,6 +57,8 @@ public:
|
||||
/// This function add UsersConfigAccessStorage if it wasn't added before.
|
||||
void setUsersConfig(const Poco::Util::AbstractConfiguration & users_config_);
|
||||
|
||||
void checkNoReservedUsersInConfigs();
|
||||
|
||||
/// Adds UsersConfigAccessStorage.
|
||||
void addUsersConfigStorage(const Poco::Util::AbstractConfiguration & users_config_);
|
||||
|
||||
|
@ -90,6 +90,9 @@ bool Authentication::areCredentialsValid(const Credentials & credentials, const
|
||||
case AuthenticationType::SSL_CERTIFICATE:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse X.509 Authentication");
|
||||
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse Interserver Authentication");
|
||||
|
||||
case AuthenticationType::MAX:
|
||||
break;
|
||||
}
|
||||
@ -116,6 +119,9 @@ bool Authentication::areCredentialsValid(const Credentials & credentials, const
|
||||
case AuthenticationType::SSL_CERTIFICATE:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse X.509 Authentication");
|
||||
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse Interserver Authentication");
|
||||
|
||||
case AuthenticationType::MAX:
|
||||
break;
|
||||
}
|
||||
@ -146,6 +152,9 @@ bool Authentication::areCredentialsValid(const Credentials & credentials, const
|
||||
case AuthenticationType::SSL_CERTIFICATE:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse X.509 Authentication");
|
||||
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse Interserver Authentication");
|
||||
|
||||
case AuthenticationType::MAX:
|
||||
break;
|
||||
}
|
||||
@ -168,6 +177,9 @@ bool Authentication::areCredentialsValid(const Credentials & credentials, const
|
||||
case AuthenticationType::SSL_CERTIFICATE:
|
||||
return auth_data.getSSLCertificateCommonNames().contains(ssl_certificate_credentials->getCommonName());
|
||||
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
throw Authentication::Require<BasicCredentials>("ClickHouse Interserver Authentication");
|
||||
|
||||
case AuthenticationType::MAX:
|
||||
break;
|
||||
}
|
||||
|
@ -64,6 +64,12 @@ const AuthenticationTypeInfo & AuthenticationTypeInfo::get(AuthenticationType ty
|
||||
static const auto info = make_info("SSL_CERTIFICATE");
|
||||
return info;
|
||||
}
|
||||
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
{
|
||||
static const auto info = make_info("INTERSERVER_SECRET");
|
||||
return info;
|
||||
}
|
||||
case AuthenticationType::MAX:
|
||||
break;
|
||||
}
|
||||
@ -119,6 +125,7 @@ void AuthenticationData::setPassword(const String & password_)
|
||||
case AuthenticationType::LDAP:
|
||||
case AuthenticationType::KERBEROS:
|
||||
case AuthenticationType::SSL_CERTIFICATE:
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
throw Exception("Cannot specify password for authentication type " + toString(type), ErrorCodes::LOGICAL_ERROR);
|
||||
|
||||
case AuthenticationType::MAX:
|
||||
@ -202,6 +209,7 @@ void AuthenticationData::setPasswordHashBinary(const Digest & hash)
|
||||
case AuthenticationType::LDAP:
|
||||
case AuthenticationType::KERBEROS:
|
||||
case AuthenticationType::SSL_CERTIFICATE:
|
||||
case AuthenticationType::INTERSERVER_SECRET:
|
||||
throw Exception("Cannot specify password binary hash for authentication type " + toString(type), ErrorCodes::LOGICAL_ERROR);
|
||||
|
||||
case AuthenticationType::MAX:
|
||||
|
@ -32,6 +32,8 @@ enum class AuthenticationType
|
||||
/// Certificates may only be trusted if 'strict' SSL mode is enabled.
|
||||
SSL_CERTIFICATE,
|
||||
|
||||
INTERSERVER_SECRET,
|
||||
|
||||
MAX,
|
||||
};
|
||||
|
||||
|
@ -376,8 +376,6 @@ void Connection::sendClusterNameAndSalt()
|
||||
|
||||
bool Connection::ping()
|
||||
{
|
||||
// LOG_TRACE(log_wrapper.get(), "Ping");
|
||||
|
||||
try
|
||||
{
|
||||
TimeoutSetter timeout_setter(*socket, sync_request_timeout, true);
|
||||
@ -819,7 +817,6 @@ std::optional<UInt64> Connection::checkPacket(size_t timeout_microseconds)
|
||||
|
||||
if (hasReadPendingData() || poll(timeout_microseconds))
|
||||
{
|
||||
// LOG_TRACE(log_wrapper.get(), "Receiving packet type");
|
||||
UInt64 packet_type;
|
||||
readVarUInt(packet_type, *in);
|
||||
|
||||
|
@ -6,6 +6,7 @@
|
||||
#include <Access/AccessControl.h>
|
||||
#include <Access/ContextAccess.h>
|
||||
#include <Access/User.h>
|
||||
#include <Core/Protocol.h>
|
||||
#include <Interpreters/Access/InterpreterSetRoleQuery.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Interpreters/executeDDLQueryOnCluster.h>
|
||||
@ -38,6 +39,9 @@ namespace
|
||||
else if (query.names->size() == 1)
|
||||
user.setName(query.names->front()->toString());
|
||||
|
||||
if (user.getName() == USER_INTERSERVER_MARKER || user.getName().empty())
|
||||
throw Exception(ErrorCodes::BAD_ARGUMENTS, "User name is reserved");
|
||||
|
||||
if (query.auth_data)
|
||||
user.auth_data = *query.auth_data;
|
||||
|
||||
|
@ -10,6 +10,7 @@
|
||||
#include <Common/setThreadName.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Interpreters/SessionLog.h>
|
||||
#include <Interpreters/Cluster.h>
|
||||
|
||||
#include <magic_enum.hpp>
|
||||
|
||||
@ -29,6 +30,7 @@ namespace ErrorCodes
|
||||
extern const int LOGICAL_ERROR;
|
||||
extern const int SESSION_NOT_FOUND;
|
||||
extern const int SESSION_IS_LOCKED;
|
||||
extern const int AUTHENTICATION_FAILED;
|
||||
}
|
||||
|
||||
|
||||
@ -317,6 +319,7 @@ void Session::authenticate(const Credentials & credentials_, const Poco::Net::So
|
||||
}
|
||||
catch (const Exception & e)
|
||||
{
|
||||
onAuthenticationFailure(credentials_, e);
|
||||
LOG_DEBUG(log, "{} Authentication failed with error: {}", toString(auth_id), e.what());
|
||||
if (auto session_log = getSessionLog())
|
||||
session_log->addLoginFailure(auth_id, *prepared_client_info, credentials_.getUserName(), e);
|
||||
@ -327,6 +330,20 @@ void Session::authenticate(const Credentials & credentials_, const Poco::Net::So
|
||||
prepared_client_info->current_address = address;
|
||||
}
|
||||
|
||||
void Session::authenticateInterserverFake()
|
||||
{
|
||||
if (session_context)
|
||||
throw Exception("If there is a session context it must be created after authentication", ErrorCodes::LOGICAL_ERROR);
|
||||
is_internal_interserver_query = true;
|
||||
}
|
||||
|
||||
void Session::onAuthenticationFailure(const Credentials & credentials_, const Exception & e)
|
||||
{
|
||||
LOG_DEBUG(log, "{} Authentication failed with error: {}", toString(auth_id), e.what());
|
||||
if (auto session_log = getSessionLog())
|
||||
session_log->addLoginFailure(auth_id, *prepared_client_info, credentials_.getUserName(), e);
|
||||
}
|
||||
|
||||
ClientInfo & Session::getClientInfo()
|
||||
{
|
||||
return session_context ? session_context->getClientInfo() : *prepared_client_info;
|
||||
@ -435,7 +452,7 @@ std::shared_ptr<SessionLog> Session::getSessionLog() const
|
||||
|
||||
ContextMutablePtr Session::makeQueryContextImpl(const ClientInfo * client_info_to_copy, ClientInfo * client_info_to_move) const
|
||||
{
|
||||
if (!user_id)
|
||||
if (!user_id && !is_internal_interserver_query)
|
||||
throw Exception("Session context must be created after authentication", ErrorCodes::LOGICAL_ERROR);
|
||||
|
||||
/// We can create a query context either from a session context or from a global context.
|
||||
@ -491,13 +508,13 @@ ContextMutablePtr Session::makeQueryContextImpl(const ClientInfo * client_info_t
|
||||
|
||||
if (!notified_session_log_about_login)
|
||||
{
|
||||
if (auto session_log = getSessionLog(); user && user_id && session_log)
|
||||
if (auto session_log = getSessionLog())
|
||||
{
|
||||
session_log->addLoginSuccess(
|
||||
auth_id,
|
||||
named_session ? std::optional<std::string>(named_session->key.second) : std::nullopt,
|
||||
*query_context,
|
||||
*user);
|
||||
user);
|
||||
|
||||
notified_session_log_about_login = true;
|
||||
}
|
||||
|
@ -51,6 +51,12 @@ public:
|
||||
void authenticate(const String & user_name, const String & password, const Poco::Net::SocketAddress & address);
|
||||
void authenticate(const Credentials & credentials_, const Poco::Net::SocketAddress & address_);
|
||||
|
||||
/// Special method for authentication through "interserver secret" without a user
|
||||
void authenticateInterserverFake();
|
||||
|
||||
/// Writes a row about login failure into session log (if enabled)
|
||||
void onAuthenticationFailure(const Credentials & credentials_, const Exception & e);
|
||||
|
||||
/// Returns a reference to session ClientInfo.
|
||||
ClientInfo & getClientInfo();
|
||||
const ClientInfo & getClientInfo() const;
|
||||
@ -86,6 +92,7 @@ private:
|
||||
|
||||
mutable UserPtr user;
|
||||
std::optional<UUID> user_id;
|
||||
bool is_internal_interserver_query = false;
|
||||
|
||||
ContextMutablePtr session_context;
|
||||
mutable bool query_context_created = false;
|
||||
|
@ -4,6 +4,7 @@
|
||||
#include <Access/User.h>
|
||||
#include <Access/EnabledRolesInfo.h>
|
||||
#include <Core/Settings.h>
|
||||
#include <Core/Protocol.h>
|
||||
#include <DataTypes/DataTypeArray.h>
|
||||
#include <DataTypes/DataTypeDateTime64.h>
|
||||
#include <DataTypes/DataTypeDate.h>
|
||||
@ -94,10 +95,11 @@ NamesAndTypesList SessionLogElement::getNamesAndTypes()
|
||||
AUTH_TYPE_NAME_AND_VALUE(AuthType::SHA256_PASSWORD),
|
||||
AUTH_TYPE_NAME_AND_VALUE(AuthType::DOUBLE_SHA1_PASSWORD),
|
||||
AUTH_TYPE_NAME_AND_VALUE(AuthType::LDAP),
|
||||
AUTH_TYPE_NAME_AND_VALUE(AuthType::KERBEROS)
|
||||
AUTH_TYPE_NAME_AND_VALUE(AuthType::KERBEROS),
|
||||
AUTH_TYPE_NAME_AND_VALUE(AuthType::INTERSERVER_SECRET),
|
||||
});
|
||||
#undef AUTH_TYPE_NAME_AND_VALUE
|
||||
static_assert(static_cast<int>(AuthenticationType::MAX) == 7);
|
||||
static_assert(static_cast<int>(AuthenticationType::MAX) == 8);
|
||||
|
||||
auto interface_type_column = std::make_shared<DataTypeEnum8>(
|
||||
DataTypeEnum8::Values
|
||||
@ -207,7 +209,7 @@ void SessionLogElement::appendToBlock(MutableColumns & columns) const
|
||||
columns[i++]->insertData(auth_failure_reason.data(), auth_failure_reason.length());
|
||||
}
|
||||
|
||||
void SessionLog::addLoginSuccess(const UUID & auth_id, std::optional<String> session_id, const Context & login_context, const User & login_user)
|
||||
void SessionLog::addLoginSuccess(const UUID & auth_id, std::optional<String> session_id, const Context & login_context, const UserPtr & login_user)
|
||||
{
|
||||
const auto access = login_context.getAccess();
|
||||
const auto & settings = login_context.getSettingsRef();
|
||||
@ -216,9 +218,9 @@ void SessionLog::addLoginSuccess(const UUID & auth_id, std::optional<String> ses
|
||||
DB::SessionLogElement log_entry(auth_id, SESSION_LOGIN_SUCCESS);
|
||||
log_entry.client_info = client_info;
|
||||
|
||||
log_entry.user = login_user.getName();
|
||||
log_entry.user_identified_with = login_user.auth_data.getType();
|
||||
log_entry.external_auth_server = login_user.auth_data.getLDAPServerName();
|
||||
log_entry.user = login_user ? login_user->getName() : USER_INTERSERVER_MARKER;
|
||||
log_entry.user_identified_with = login_user ? login_user->auth_data.getType() : AuthenticationType::INTERSERVER_SECRET;
|
||||
log_entry.external_auth_server = login_user ? login_user->auth_data.getLDAPServerName() : "";
|
||||
|
||||
if (session_id)
|
||||
log_entry.session_id = *session_id;
|
||||
|
@ -19,6 +19,7 @@ enum SessionLogElementType : int8_t
|
||||
|
||||
class ContextAccess;
|
||||
struct User;
|
||||
using UserPtr = std::shared_ptr<const User>;
|
||||
|
||||
/** A struct which will be inserted as row into session_log table.
|
||||
*
|
||||
@ -71,7 +72,7 @@ class SessionLog : public SystemLog<SessionLogElement>
|
||||
using SystemLog<SessionLogElement>::SystemLog;
|
||||
|
||||
public:
|
||||
void addLoginSuccess(const UUID & auth_id, std::optional<String> session_id, const Context & login_context, const User & login_user);
|
||||
void addLoginSuccess(const UUID & auth_id, std::optional<String> session_id, const Context & login_context, const UserPtr & login_user);
|
||||
void addLoginFailure(const UUID & auth_id, const ClientInfo & info, const String & user, const Exception & reason);
|
||||
void addLogOut(const UUID & auth_id, const String & user, const ClientInfo & client_info);
|
||||
};
|
||||
|
@ -95,6 +95,7 @@ namespace
|
||||
}
|
||||
|
||||
case AuthenticationType::NO_PASSWORD: [[fallthrough]];
|
||||
case AuthenticationType::INTERSERVER_SECRET: [[fallthrough]];
|
||||
case AuthenticationType::MAX:
|
||||
throw Exception("AST: Unexpected authentication type " + toString(auth_type), ErrorCodes::LOGICAL_ERROR);
|
||||
}
|
||||
|
@ -79,6 +79,7 @@ namespace ErrorCodes
|
||||
extern const int UNEXPECTED_PACKET_FROM_CLIENT;
|
||||
extern const int SUPPORT_IS_DISABLED;
|
||||
extern const int UNKNOWN_PROTOCOL;
|
||||
extern const int AUTHENTICATION_FAILED;
|
||||
}
|
||||
|
||||
TCPHandler::TCPHandler(IServer & server_, TCPServer & tcp_server_, const Poco::Net::StreamSocket & socket_, bool parse_proxy_protocol_, std::string server_display_name_)
|
||||
@ -1210,25 +1211,6 @@ void TCPHandler::receiveClusterNameAndSalt()
|
||||
{
|
||||
readStringBinary(cluster, *in);
|
||||
readStringBinary(salt, *in, 32);
|
||||
|
||||
try
|
||||
{
|
||||
if (salt.empty())
|
||||
throw NetException("Empty salt is not allowed", ErrorCodes::UNEXPECTED_PACKET_FROM_CLIENT);
|
||||
|
||||
cluster_secret = server.context()->getCluster(cluster)->getSecret();
|
||||
}
|
||||
catch (const Exception & e)
|
||||
{
|
||||
try
|
||||
{
|
||||
/// We try to send error information to the client.
|
||||
sendException(e, send_exception_with_stack_trace);
|
||||
}
|
||||
catch (...) {}
|
||||
|
||||
throw;
|
||||
}
|
||||
}
|
||||
|
||||
void TCPHandler::receiveQuery()
|
||||
@ -1239,7 +1221,7 @@ void TCPHandler::receiveQuery()
|
||||
state.is_empty = false;
|
||||
readStringBinary(state.query_id, *in);
|
||||
|
||||
/// In interserer mode,
|
||||
/// In interserver mode,
|
||||
/// initial_user can be empty in case of Distributed INSERT via Buffer/Kafka,
|
||||
/// (i.e. when the INSERT is done with the global context without user),
|
||||
/// so it is better to reset session to avoid using old user.
|
||||
@ -1285,27 +1267,41 @@ void TCPHandler::receiveQuery()
|
||||
|
||||
readStringBinary(state.query, *in);
|
||||
|
||||
/// TODO unify interserver authentication (currently this code looks like a backdoor at a first glance)
|
||||
if (is_interserver_mode)
|
||||
{
|
||||
#if USE_SSL
|
||||
String user_for_session_log = client_info.initial_user;
|
||||
if (user_for_session_log.empty())
|
||||
user_for_session_log = USER_INTERSERVER_MARKER;
|
||||
if (salt.empty())
|
||||
{
|
||||
auto exception = Exception(ErrorCodes::AUTHENTICATION_FAILED, "Interserver authentication failed");
|
||||
session->onAuthenticationFailure(AlwaysAllowCredentials{USER_INTERSERVER_MARKER}, exception);
|
||||
throw exception;
|
||||
}
|
||||
|
||||
std::string data(salt);
|
||||
data += cluster_secret;
|
||||
data += server.context()->getCluster(cluster)->getSecret();
|
||||
data += state.query;
|
||||
data += state.query_id;
|
||||
data += client_info.initial_user;
|
||||
|
||||
if (received_hash.size() != 32)
|
||||
throw NetException("Unexpected hash received from client", ErrorCodes::UNEXPECTED_PACKET_FROM_CLIENT);
|
||||
|
||||
std::string calculated_hash = encodeSHA256(data);
|
||||
assert(calculated_hash.size() == 32);
|
||||
|
||||
/// TODO maybe also check that peer address actually belongs to the cluster?
|
||||
if (calculated_hash != received_hash)
|
||||
throw NetException("Hash mismatch", ErrorCodes::UNEXPECTED_PACKET_FROM_CLIENT);
|
||||
/// TODO: change error code?
|
||||
{
|
||||
auto exception = Exception(ErrorCodes::AUTHENTICATION_FAILED, "Interserver authentication failed");
|
||||
session->onAuthenticationFailure(AlwaysAllowCredentials{USER_INTERSERVER_MARKER}, exception);
|
||||
throw exception;
|
||||
}
|
||||
|
||||
if (client_info.initial_user.empty())
|
||||
{
|
||||
LOG_DEBUG(log, "User (no user, interserver mode)");
|
||||
session->authenticateInterserverFake();
|
||||
}
|
||||
else
|
||||
{
|
||||
@ -1313,9 +1309,11 @@ void TCPHandler::receiveQuery()
|
||||
session->authenticate(AlwaysAllowCredentials{client_info.initial_user}, client_info.initial_address);
|
||||
}
|
||||
#else
|
||||
throw Exception(
|
||||
auto exception = Exception(
|
||||
"Inter-server secret support is disabled, because ClickHouse was built without SSL library",
|
||||
ErrorCodes::SUPPORT_IS_DISABLED);
|
||||
ErrorCodes::AUTHENTICATION_FAILED);
|
||||
session->onAuthenticationFailure(AlwaysAllowCredentials{USER_INTERSERVER_MARKER}, exception);
|
||||
throw exception;
|
||||
#endif
|
||||
}
|
||||
|
||||
|
@ -182,7 +182,6 @@ private:
|
||||
bool is_interserver_mode = false;
|
||||
String salt;
|
||||
String cluster;
|
||||
String cluster_secret;
|
||||
|
||||
std::mutex task_callback_mutex;
|
||||
std::mutex fatal_error_mutex;
|
||||
|
@ -291,20 +291,20 @@ def test_secure_insert_buffer_async():
|
||||
|
||||
|
||||
def test_secure_disagree():
|
||||
with pytest.raises(QueryRuntimeException, match=".*Hash mismatch.*"):
|
||||
with pytest.raises(QueryRuntimeException, match=".*Interserver authentication failed.*"):
|
||||
n1.query("SELECT * FROM dist_secure_disagree")
|
||||
|
||||
|
||||
def test_secure_disagree_insert():
|
||||
n1.query("TRUNCATE TABLE data")
|
||||
n1.query("INSERT INTO dist_secure_disagree SELECT * FROM numbers(2)")
|
||||
with pytest.raises(QueryRuntimeException, match=".*Hash mismatch.*"):
|
||||
with pytest.raises(QueryRuntimeException, match=".*Interserver authentication failed.*"):
|
||||
n1.query(
|
||||
"SYSTEM FLUSH DISTRIBUTED ON CLUSTER secure_disagree dist_secure_disagree"
|
||||
)
|
||||
# check the the connection will be re-established
|
||||
# IOW that we will not get "Unknown BlockInfo field"
|
||||
with pytest.raises(QueryRuntimeException, match=".*Hash mismatch.*"):
|
||||
with pytest.raises(QueryRuntimeException, match=".*Interserver authentication failed.*"):
|
||||
assert int(n1.query("SELECT count() FROM dist_secure_disagree")) == 0
|
||||
|
||||
|
||||
|
Loading…
Reference in New Issue
Block a user