io.streamnative.pulsar.handlers.kop.SchemaRegistryManager Maven / Gradle / Ivy
Go to download
Show more of this group Show more artifacts with this name
Show all versions of pulsar-protocol-handler-kafka Show documentation
Show all versions of pulsar-protocol-handler-kafka Show documentation
Kafka on Pulsar implemented using Pulsar Protocol Handler
The newest version!
/**
* Copyright (c) 2019 - 2024 StreamNative, Inc.. All Rights Reserved.
*/
/**
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package io.streamnative.pulsar.handlers.kop;
import static io.streamnative.pulsar.handlers.kop.KafkaProtocolHandler.TLS_HANDLER;
import com.google.common.annotations.VisibleForTesting;
import com.google.common.base.Strings;
import io.netty.channel.ChannelHandler;
import io.netty.channel.ChannelHandlerContext;
import io.netty.channel.ChannelPipeline;
import io.netty.handler.codec.http.FullHttpRequest;
import io.netty.handler.codec.http.HttpHeaderNames;
import io.netty.handler.codec.http.HttpResponseStatus;
import io.netty.handler.ssl.SslHandler;
import io.streamnative.pulsar.handlers.kop.schemaregistry.DummyOptionsCORSProcessor;
import io.streamnative.pulsar.handlers.kop.schemaregistry.SchemaRegistryChannelInitializer;
import io.streamnative.pulsar.handlers.kop.schemaregistry.SchemaRegistryHandler;
import io.streamnative.pulsar.handlers.kop.schemaregistry.SchemaRegistryRequestAuthenticator;
import io.streamnative.pulsar.handlers.kop.schemaregistry.SchemaRegistryStats;
import io.streamnative.pulsar.handlers.kop.schemaregistry.model.SchemaStorageAccessor;
import io.streamnative.pulsar.handlers.kop.schemaregistry.model.impl.OxiaSchemaStorageAccessor;
import io.streamnative.pulsar.handlers.kop.schemaregistry.model.impl.PulsarSchemaStorageAccessor;
import io.streamnative.pulsar.handlers.kop.schemaregistry.model.impl.SchemaStorageException;
import io.streamnative.pulsar.handlers.kop.schemaregistry.resources.CompatibilityResource;
import io.streamnative.pulsar.handlers.kop.schemaregistry.resources.ConfigResource;
import io.streamnative.pulsar.handlers.kop.schemaregistry.resources.SchemaResource;
import io.streamnative.pulsar.handlers.kop.schemaregistry.resources.SubjectResource;
import io.streamnative.pulsar.handlers.kop.schemaregistry.resources.SubjectVersionsResource;
import io.streamnative.pulsar.handlers.kop.security.KafkaPrincipal;
import io.streamnative.pulsar.handlers.kop.security.auth.Authorizer;
import io.streamnative.pulsar.handlers.kop.security.auth.Resource;
import io.streamnative.pulsar.handlers.kop.security.auth.ResourceType;
import io.streamnative.pulsar.handlers.kop.security.auth.SimpleAclAuthorizer;
import io.streamnative.pulsar.handlers.kop.security.oauth.OAuthTokenDecoder;
import io.streamnative.pulsar.handlers.kop.utils.MetadataUtils;
import java.net.InetSocketAddress;
import java.net.SocketAddress;
import java.nio.charset.StandardCharsets;
import java.util.ArrayList;
import java.util.Base64;
import java.util.Optional;
import java.util.concurrent.CompletableFuture;
import java.util.concurrent.ExecutionException;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.TimeoutException;
import java.util.function.Consumer;
import javax.naming.AuthenticationException;
import javax.net.ssl.SSLSession;
import lombok.AllArgsConstructor;
import lombok.Getter;
import lombok.extern.slf4j.Slf4j;
import org.apache.commons.lang.StringUtils;
import org.apache.commons.lang3.tuple.Pair;
import org.apache.pulsar.broker.PulsarService;
import org.apache.pulsar.broker.authentication.AuthenticationDataSource;
import org.apache.pulsar.broker.authentication.AuthenticationProvider;
import org.apache.pulsar.broker.authentication.AuthenticationService;
import org.apache.pulsar.broker.authentication.AuthenticationState;
import org.apache.pulsar.broker.service.BrokerService;
import org.apache.pulsar.client.admin.PulsarAdmin;
import org.apache.pulsar.client.api.PulsarClient;
import org.apache.pulsar.common.api.AuthData;
import org.apache.pulsar.common.policies.data.ClusterData;
import org.apache.pulsar.common.util.FutureUtil;
@Slf4j
public class SchemaRegistryManager {
private final KafkaServiceConfiguration kafkaConfig;
private final PulsarService pulsar;
private final SchemaRegistryRequestAuthenticator schemaRegistryRequestAuthenticator;
private final PulsarClient pulsarClient;
@Getter
@VisibleForTesting
private SchemaStorageAccessor schemaStorage;
public SchemaRegistryManager(KafkaServiceConfiguration kafkaConfig,
PulsarService pulsar,
AuthenticationService authenticationService) {
this.kafkaConfig = kafkaConfig;
this.pulsarClient = SystemTopicClient.createPulsarClient(pulsar, kafkaConfig, (___) -> {});
this.pulsar = pulsar;
Authorizer authorizer = new SimpleAclAuthorizer(pulsar, kafkaConfig);
this.schemaRegistryRequestAuthenticator = new HttpRequestAuthenticator(this.kafkaConfig,
authenticationService, authorizer);
}
private static final class AuthDataPair {
final String tenant;
final AuthData authData;
AuthDataPair(String tenant, String data) {
this.tenant = tenant;
this.authData = AuthData.of(data.getBytes(StandardCharsets.UTF_8));
}
}
@AllArgsConstructor
public static class HttpRequestAuthenticator implements SchemaRegistryRequestAuthenticator {
private final KafkaServiceConfiguration kafkaConfig;
private final AuthenticationService authenticationService;
private final Authorizer authorizer;
private static final byte[] emptyArray = new byte[0];
@Override
public String authenticate(FullHttpRequest request, ChannelHandlerContext ctx) throws SchemaStorageException {
if (!kafkaConfig.isAuthenticationEnabled()) {
return kafkaConfig.getKafkaMetadataTenant();
}
ChannelHandler tlsHandler = ctx.channel().pipeline().get(TLS_HANDLER);
AuthenticationProvider tlsAuthProvider = authenticationService
.getAuthenticationProvider("tls");
if (tlsHandler != null && tlsAuthProvider != null) {
SSLSession sslSession = ((SslHandler) tlsHandler).engine().getSession();
AuthData authData = AuthData.of(emptyArray);
SocketAddress remoteAddress = ctx.channel().remoteAddress();
try {
AuthenticationState authState = tlsAuthProvider.newAuthState(authData, remoteAddress, sslSession);
authState.authenticateAsync(authData)
.get(kafkaConfig.getRequestTimeoutMs(), TimeUnit.MILLISECONDS);
final String role = authState.getAuthRole();
final String tenant = kafkaConfig.getKafkaMetadataTenant();
performAuthorizationValidation(null, role, authState.getAuthDataSource(), tenant);
return tenant;
} catch (ExecutionException | InterruptedException | TimeoutException | AuthenticationException err) {
throw new SchemaStorageException(err);
}
}
try {
AuthenticationProvider authenticationProvider = authenticationService
.getAuthenticationProvider("token");
if (authenticationProvider == null) {
throw new SchemaStorageException("Pulsar is not configured for Token auth");
}
String authenticationHeader = request.headers().get(HttpHeaderNames.AUTHORIZATION, "");
AuthDataPair authDataPair = parseAuthHeader(authenticationHeader);
final AuthenticationState authState = authenticationProvider
.newAuthState(authDataPair.authData, null, null);
authState.authenticateAsync(authDataPair.authData)
.get(kafkaConfig.getRequestTimeoutMs(), TimeUnit.MILLISECONDS);
final String role = authState.getAuthRole();
final String tenant;
if (kafkaConfig.isKafkaEnableMultiTenantMetadata()) {
// the tenant is the username
log.debug("SchemaRegistry Authenticated username {} role {} using tenant {} for data",
authDataPair.tenant, role, authDataPair.tenant);
if (StringUtils.isBlank(authDataPair.tenant)) {
throw new SchemaStorageException(
"Tenant must be provided when multi-tenant metadata enabled");
}
tenant = authDataPair.tenant;
} else {
// use system tenant
log.debug("SchemaRegistry Authenticated username {} role {} using system tenant {} for data",
authDataPair.tenant, role, kafkaConfig.getKafkaMetadataTenant());
tenant = kafkaConfig.getKafkaMetadataTenant();
}
performAuthorizationValidation(authDataPair.tenant, role, authState.getAuthDataSource(), tenant);
return tenant;
} catch (ExecutionException
| InterruptedException
| TimeoutException
| IllegalArgumentException
| AuthenticationException err) {
Throwable throwable = FutureUtil.unwrapCompletionException(err);
if (throwable instanceof AuthenticationException) {
throw new SchemaStorageException(throwable.getMessage(), HttpResponseStatus.UNAUTHORIZED);
}
if (throwable instanceof IllegalArgumentException) {
throw new SchemaStorageException(throwable.getMessage(), HttpResponseStatus.UNAUTHORIZED);
}
throw new SchemaStorageException(err);
}
}
private void performAuthorizationValidation(String username, String role,
AuthenticationDataSource authData, String tenant)
throws SchemaStorageException {
if (!kafkaConfig.isAuthorizationEnabled()) {
return;
}
KafkaPrincipal kafkaPrincipal =
new KafkaPrincipal(KafkaPrincipal.USER_TYPE, role, username, null, authData);
String topicName = MetadataUtils.constructSchemaRegistryTopicName(tenant, kafkaConfig);
if (kafkaConfig.isKafkaEnableMultiTenantMetadata()) {
tenantAuthorizationCheck(kafkaPrincipal, tenant, topicName, username, role);
}
schemaRegistryTopicAuthorizationCheck(kafkaPrincipal, tenant, topicName, username, role);
}
private void tenantAuthorizationCheck(KafkaPrincipal kafkaPrincipal, String tenant, String topicName,
String username, String role) throws SchemaStorageException {
try {
Boolean tenantExists =
authorizer.canAccessTenantAsync(kafkaPrincipal, Resource.of(ResourceType.TENANT, tenant))
.get();
if (tenantExists == null || !tenantExists) {
log.debug("SchemaRegistry username {} role {} tenant {} does not exist, cannot access topic {}",
username, role, tenant, topicName);
throw new SchemaStorageException("Role " + role + " cannot access topic " + topicName + " "
+ "tenant " + tenant + " does not exist (wrong username?)",
HttpResponseStatus.FORBIDDEN);
}
} catch (ExecutionException | InterruptedException err) {
handleAuthorizationException(err);
}
}
private void schemaRegistryTopicAuthorizationCheck(KafkaPrincipal kafkaPrincipal, String tenant,
String topicName, String username, String role)
throws SchemaStorageException {
try {
Boolean hasPermission = authorizer
.canProduceAsync(kafkaPrincipal, Resource.of(ResourceType.TOPIC, topicName))
.get();
if (hasPermission == null || !hasPermission) {
log.debug("SchemaRegistry username {} role {} tenant {} cannot access topic {}",
username, role, tenant, topicName);
throw new SchemaStorageException("Role " + role + " cannot access topic " + topicName,
HttpResponseStatus.FORBIDDEN);
}
} catch (ExecutionException | InterruptedException err) {
handleAuthorizationException(err);
}
}
private void handleAuthorizationException(Exception err) throws SchemaStorageException {
if (err instanceof ExecutionException) {
throw new SchemaStorageException(err.getCause());
} else if (err instanceof InterruptedException) {
throw new SchemaStorageException(err);
}
}
private AuthDataPair parseAuthHeader(String authenticationHeader)
throws SchemaStorageException {
if (authenticationHeader.isEmpty()) {
// no auth
throw new SchemaStorageException("Missing AUTHORIZATION header",
HttpResponseStatus.UNAUTHORIZED);
}
boolean isBasic = authenticationHeader.startsWith("Basic ");
boolean isBearer = authenticationHeader.startsWith("Bearer ");
if (!isBasic && !isBearer) {
throw new SchemaStorageException("Bad authentication scheme, neither Basic nor Bearer.",
HttpResponseStatus.UNAUTHORIZED);
}
String strippedAuthHeader = authenticationHeader.substring(
isBasic ? "Basic ".length() : "Bearer ".length());
return isBasic ? parseBasicAuthHeader(strippedAuthHeader) : parseBearerAuthHeader(strippedAuthHeader);
}
private AuthDataPair parseBasicAuthHeader(String originalAuthHeader) throws SchemaStorageException {
String authHeader = new String(Base64.getDecoder()
.decode(originalAuthHeader), StandardCharsets.UTF_8);
int colon = authHeader.indexOf(":");
if (colon <= 0) {
throw new SchemaStorageException("Bad authentication header", HttpResponseStatus.BAD_REQUEST);
}
String rawUsername = authHeader.substring(0, colon);
String rawPassword = authHeader.substring(colon + 1);
if (rawPassword.startsWith("token:")) {
return new AuthDataPair(rawUsername, rawPassword.substring("token:".length()));
} else {
return new AuthDataPair(rawUsername, rawPassword);
}
}
private AuthDataPair parseBearerAuthHeader(String authHeader) {
Pair tokenAndTenant = OAuthTokenDecoder.decode(authHeader);
return new AuthDataPair(tokenAndTenant.getRight(), tokenAndTenant.getLeft());
}
}
public InetSocketAddress getAddress() {
return new InetSocketAddress(kafkaConfig.getKopSchemaRegistryPort());
}
public Optional build(Consumer pipelineCustomizer,
SchemaRegistryStats stats)
throws Exception {
if (!kafkaConfig.isKopSchemaRegistryEnable()) {
return Optional.empty();
}
PulsarAdmin pulsarAdmin = pulsar.getAdminClient();
SchemaRegistryHandler handler = new SchemaRegistryHandler(schemaRegistryRequestAuthenticator, stats);
if (kafkaConfig.isKafkaSchemaRegistryStoreInOxia()) {
String oxiaSchemaRegistryUrl = kafkaConfig.getOxiaSchemaRegistryUrl();
if (Strings.isNullOrEmpty(oxiaSchemaRegistryUrl)) {
oxiaSchemaRegistryUrl = kafkaConfig.getMetadataStoreUrl();
}
schemaStorage = new OxiaSchemaStorageAccessor(oxiaSchemaRegistryUrl);
} else {
schemaStorage = new PulsarSchemaStorageAccessor((tenant) -> {
try {
BrokerService brokerService = pulsar.getBrokerService();
final ClusterData clusterData = ClusterData.builder()
.serviceUrl(brokerService.getPulsar().getWebServiceAddress())
.serviceUrlTls(brokerService.getPulsar().getWebServiceAddressTls())
.brokerServiceUrl(brokerService.getPulsar().getBrokerServiceUrl())
.brokerServiceUrlTls(brokerService.getPulsar().getBrokerServiceUrlTls())
.build();
MetadataUtils.createSchemaRegistryMetadataIfMissing(tenant,
pulsarAdmin,
clusterData,
kafkaConfig);
return pulsarClient;
} catch (Exception err) {
throw new IllegalStateException(err);
}
}, kafkaConfig.getKopSchemaRegistryNamespace(), kafkaConfig.getKopSchemaRegistryTopicName(), stats,
pulsar.getOrderedExecutor());
}
final String defaultNamespace = kafkaConfig.getKafkaTenant() + "/" + kafkaConfig.getKafkaNamespace();
new SchemaResource(schemaStorage, defaultNamespace).register(handler);
new SubjectVersionsResource(schemaStorage, defaultNamespace)
.register(handler);
new SubjectResource(schemaStorage, defaultNamespace).register(handler);
new ConfigResource(schemaStorage, defaultNamespace).register(handler);
new CompatibilityResource(schemaStorage, defaultNamespace)
.register(handler);
handler.addProcessor(new DummyOptionsCORSProcessor());
return Optional.of(new SchemaRegistryChannelInitializer(handler, pipelineCustomizer));
}
public CompletableFuture closeAsync() {
final var futures = new ArrayList>();
futures.add(pulsarClient.closeAsync());
if (schemaStorage != null) {
futures.add(schemaStorage.closeAsync());
}
return FutureUtil.waitForAll(futures);
}
}