Skip to content

Commit 2f856db

Browse files
nodecesrinath-ctds
authored andcommitted
[fix][client][branch-3.0] Fix compatibility between kerberos and tls (apache#23801)
Signed-off-by: Zixuan Liu <[email protected]> (cherry picked from commit 4b69b30)
1 parent b22764f commit 2f856db

File tree

12 files changed

+227
-140
lines changed

12 files changed

+227
-140
lines changed

pulsar-broker/src/test/java/org/apache/pulsar/client/api/TlsProducerConsumerTest.java

+69
Original file line numberDiff line numberDiff line change
@@ -18,18 +18,24 @@
1818
*/
1919
package org.apache.pulsar.client.api;
2020

21+
import static org.mockito.Mockito.mock;
22+
import static org.mockito.Mockito.never;
23+
import static org.mockito.Mockito.spy;
24+
import static org.mockito.Mockito.verify;
2125
import java.io.ByteArrayInputStream;
2226
import java.io.ByteArrayOutputStream;
2327
import java.io.FileInputStream;
2428
import java.io.IOException;
2529
import java.io.InputStream;
2630
import java.util.Arrays;
31+
import java.util.Map;
2732
import java.util.concurrent.TimeUnit;
2833
import java.util.concurrent.atomic.AtomicInteger;
2934
import java.util.function.Supplier;
3035
import lombok.Cleanup;
3136
import org.apache.commons.compress.utils.IOUtils;
3237
import org.apache.pulsar.broker.service.persistent.PersistentTopic;
38+
import org.apache.pulsar.client.admin.PulsarAdmin;
3339
import org.apache.pulsar.client.impl.auth.AuthenticationTls;
3440
import org.slf4j.Logger;
3541
import org.slf4j.LoggerFactory;
@@ -293,4 +299,67 @@ public void testTlsTransport(Supplier<String> url, Authentication auth) throws E
293299
@Cleanup
294300
Producer<byte[]> ignored = client.newProducer().topic(topicName).create();
295301
}
302+
303+
@Test
304+
public void testTlsWithFakeAuthentication() throws Exception {
305+
Authentication authentication = spy(new Authentication() {
306+
@Override
307+
public String getAuthMethodName() {
308+
return "fake";
309+
}
310+
311+
@Override
312+
public void configure(Map<String, String> authParams) {
313+
314+
}
315+
316+
@Override
317+
public void start() {
318+
319+
}
320+
321+
@Override
322+
public void close() {
323+
324+
}
325+
326+
@Override
327+
public AuthenticationDataProvider getAuthData(String brokerHostName) {
328+
return mock(AuthenticationDataProvider.class);
329+
}
330+
});
331+
332+
@Cleanup
333+
PulsarAdmin pulsarAdmin = PulsarAdmin.builder()
334+
.serviceHttpUrl(getPulsar().getWebServiceAddressTls())
335+
.tlsTrustCertsFilePath(CA_CERT_FILE_PATH)
336+
.allowTlsInsecureConnection(false)
337+
.enableTlsHostnameVerification(false)
338+
.tlsKeyFilePath(getTlsFileForClient("admin.key-pk8"))
339+
.tlsCertificateFilePath(getTlsFileForClient("admin.cert"))
340+
.authentication(authentication)
341+
.build();
342+
pulsarAdmin.tenants().getTenants();
343+
verify(authentication, never()).getAuthData();
344+
345+
@Cleanup
346+
PulsarClient pulsarClient = PulsarClient.builder().serviceUrl(getPulsar().getBrokerServiceUrlTls())
347+
.tlsTrustCertsFilePath(CA_CERT_FILE_PATH)
348+
.allowTlsInsecureConnection(false)
349+
.enableTlsHostnameVerification(false)
350+
.tlsKeyFilePath(getTlsFileForClient("admin.key-pk8"))
351+
.tlsCertificateFilePath(getTlsFileForClient("admin.cert"))
352+
.authentication(authentication).build();
353+
verify(authentication, never()).getAuthData();
354+
355+
final String topicName = "persistent://my-property/my-ns/my-topic-1";
356+
internalSetUpForNamespace();
357+
@Cleanup
358+
Consumer<byte[]> ignoredConsumer =
359+
pulsarClient.newConsumer().topic(topicName).subscriptionName("my-subscriber-name").subscribe();
360+
verify(authentication, never()).getAuthData();
361+
@Cleanup
362+
Producer<byte[]> ignoredProducer = pulsarClient.newProducer().topic(topicName).create();
363+
verify(authentication, never()).getAuthData();
364+
}
296365
}

pulsar-client-admin/src/main/java/org/apache/pulsar/client/admin/internal/http/AsyncHttpConnector.java

+2-1
Original file line numberDiff line numberDiff line change
@@ -186,7 +186,8 @@ private void configureAsyncHttpClientSslEngineFactory(ClientConfigurationData co
186186
DefaultAsyncHttpClientConfig.Builder confBuilder)
187187
throws GeneralSecurityException, IOException {
188188
// Set client key and certificate if available
189-
AuthenticationDataProvider authData = conf.getAuthentication().getAuthData();
189+
AuthenticationDataProvider authData = conf.getAuthentication().getAuthData(serviceNameResolver
190+
.resolveHostUri().getHost());
190191

191192
SslEngineFactory sslEngineFactory = null;
192193
if (conf.isUseKeyStoreTls()) {

pulsar-client-api/src/main/java/org/apache/pulsar/client/api/Authentication.java

+1
Original file line numberDiff line numberDiff line change
@@ -48,6 +48,7 @@ public interface Authentication extends Closeable, Serializable {
4848
* @throws PulsarClientException
4949
* any other error
5050
*/
51+
@Deprecated
5152
default AuthenticationDataProvider getAuthData() throws PulsarClientException {
5253
throw new UnsupportedAuthenticationException("Method not implemented!");
5354
}

pulsar-client-tools/src/main/java/org/apache/pulsar/client/cli/CmdConsume.java

+1-1
Original file line numberDiff line numberDiff line change
@@ -254,7 +254,7 @@ private int consumeFromWebSocket(String topic) {
254254
try {
255255
if (authentication != null) {
256256
authentication.start();
257-
AuthenticationDataProvider authData = authentication.getAuthData();
257+
AuthenticationDataProvider authData = authentication.getAuthData(consumerUri.getHost());
258258
if (authData.hasDataForHttp()) {
259259
for (Map.Entry<String, String> kv : authData.getHttpHeaders()) {
260260
consumeRequest.setHeader(kv.getKey(), kv.getValue());

pulsar-client-tools/src/main/java/org/apache/pulsar/client/cli/CmdProduce.java

+1-1
Original file line numberDiff line numberDiff line change
@@ -469,7 +469,7 @@ private int publishToWebSocket(String topic) {
469469
try {
470470
if (authentication != null) {
471471
authentication.start();
472-
AuthenticationDataProvider authData = authentication.getAuthData();
472+
AuthenticationDataProvider authData = authentication.getAuthData(produceUri.getHost());
473473
if (authData.hasDataForHttp()) {
474474
for (Map.Entry<String, String> kv : authData.getHttpHeaders()) {
475475
produceRequest.setHeader(kv.getKey(), kv.getValue());

pulsar-client-tools/src/main/java/org/apache/pulsar/client/cli/CmdRead.java

+1-1
Original file line numberDiff line numberDiff line change
@@ -243,7 +243,7 @@ private int readFromWebSocket(String topic) {
243243
try {
244244
if (authentication != null) {
245245
authentication.start();
246-
AuthenticationDataProvider authData = authentication.getAuthData();
246+
AuthenticationDataProvider authData = authentication.getAuthData(readerUri.getHost());
247247
if (authData.hasDataForHttp()) {
248248
for (Map.Entry<String, String> kv : authData.getHttpHeaders()) {
249249
readRequest.setHeader(kv.getKey(), kv.getValue());

pulsar-client/src/main/java/org/apache/pulsar/client/impl/HttpClient.java

+2-1
Original file line numberDiff line numberDiff line change
@@ -93,7 +93,8 @@ public boolean keepAlive(InetSocketAddress remoteAddress, Request ahcRequest,
9393
if ("https".equals(serviceNameResolver.getServiceUri().getServiceName())) {
9494
try {
9595
// Set client key and certificate if available
96-
AuthenticationDataProvider authData = authentication.getAuthData();
96+
AuthenticationDataProvider authData =
97+
authentication.getAuthData(serviceNameResolver.resolveHostUri().getHost());
9798

9899
if (conf.isUseKeyStoreTls()) {
99100
SSLContext sslCtx = null;

pulsar-client/src/main/java/org/apache/pulsar/client/impl/PulsarChannelInitializer.java

+78-68
Original file line numberDiff line numberDiff line change
@@ -29,8 +29,10 @@
2929
import io.netty.handler.ssl.SslHandler;
3030
import io.netty.handler.ssl.SslProvider;
3131
import java.net.InetSocketAddress;
32+
import java.util.Map;
3233
import java.util.Objects;
3334
import java.util.concurrent.CompletableFuture;
35+
import java.util.concurrent.ConcurrentHashMap;
3436
import java.util.concurrent.TimeUnit;
3537
import java.util.function.Supplier;
3638
import lombok.Getter;
@@ -59,9 +61,9 @@ public class PulsarChannelInitializer extends ChannelInitializer<SocketChannel>
5961
private final InetSocketAddress socks5ProxyAddress;
6062
private final String socks5ProxyUsername;
6163
private final String socks5ProxyPassword;
62-
63-
private final Supplier<SslContext> sslContextSupplier;
64-
private NettySSLContextAutoRefreshBuilder nettySSLContextAutoRefreshBuilder;
64+
private final ClientConfigurationData conf;
65+
private Map<String, Supplier<SslContext>> sslContextSupplierMap;
66+
private Map<String, NettySSLContextAutoRefreshBuilder> nettySSLContextAutoRefreshBuilderMap;
6567

6668
private static final long TLS_CERTIFICATE_CACHE_MILLIS = TimeUnit.MINUTES.toMillis(1);
6769

@@ -76,15 +78,34 @@ public PulsarChannelInitializer(ClientConfigurationData conf, Supplier<ClientCnx
7678
this.socks5ProxyPassword = conf.getSocks5ProxyPassword();
7779

7880
this.tlsEnabledWithKeyStore = conf.isUseKeyStoreTls();
81+
this.conf = conf.clone();
82+
this.sslContextSupplierMap = new ConcurrentHashMap<>();
83+
this.nettySSLContextAutoRefreshBuilderMap = new ConcurrentHashMap<>();
84+
}
7985

80-
if (tlsEnabled) {
81-
if (tlsEnabledWithKeyStore) {
82-
AuthenticationDataProvider authData1 = conf.getAuthentication().getAuthData();
83-
if (StringUtils.isBlank(conf.getTlsTrustStorePath())) {
84-
throw new PulsarClientException("Failed to create TLS context, the tlsTrustStorePath"
85-
+ " need to be configured if useKeyStoreTls enabled");
86-
}
87-
nettySSLContextAutoRefreshBuilder = new NettySSLContextAutoRefreshBuilder(
86+
@Override
87+
public void initChannel(SocketChannel ch) throws Exception {
88+
ch.pipeline().addLast("consolidation", new FlushConsolidationHandler(1024, true));
89+
90+
// Setup channel except for the SsHandler for TLS enabled connections
91+
ch.pipeline().addLast("ByteBufPairEncoder", ByteBufPair.getEncoder(tlsEnabled));
92+
93+
ch.pipeline().addLast("frameDecoder", new LengthFieldBasedFrameDecoder(
94+
Commands.DEFAULT_MAX_MESSAGE_SIZE + Commands.MESSAGE_SIZE_FRAME_PADDING, 0, 4, 0, 4));
95+
ChannelHandler clientCnx = clientCnxSupplier.get();
96+
ch.pipeline().addLast("handler", clientCnx);
97+
}
98+
99+
private NettySSLContextAutoRefreshBuilder getNettySSLContextAutoRefreshBuilder(String host)
100+
throws PulsarClientException {
101+
if (tlsEnabledWithKeyStore) {
102+
AuthenticationDataProvider authData1 = conf.getAuthentication().getAuthData(host);
103+
if (StringUtils.isBlank(conf.getTlsTrustStorePath())) {
104+
throw new PulsarClientException("Failed to create TLS context, the tlsTrustStorePath"
105+
+ " need to be configured if useKeyStoreTls enabled");
106+
}
107+
return nettySSLContextAutoRefreshBuilderMap.computeIfAbsent(host,
108+
key -> new NettySSLContextAutoRefreshBuilder(
88109
conf.getSslProvider(),
89110
conf.isTlsAllowInsecureConnection(),
90111
conf.getTlsTrustStoreType(),
@@ -96,64 +117,52 @@ public PulsarChannelInitializer(ClientConfigurationData conf, Supplier<ClientCnx
96117
conf.getTlsCiphers(),
97118
conf.getTlsProtocols(),
98119
TLS_CERTIFICATE_CACHE_MILLIS,
99-
authData1);
100-
}
101-
102-
sslContextSupplier = new ObjectCache<SslContext>(() -> {
103-
try {
104-
SslProvider sslProvider = null;
105-
if (conf.getSslProvider() != null) {
106-
sslProvider = SslProvider.valueOf(conf.getSslProvider());
107-
}
108-
109-
// Set client certificate if available
110-
AuthenticationDataProvider authData = conf.getAuthentication().getAuthData();
111-
if (authData.hasDataForTls()) {
112-
return authData.getTlsTrustStoreStream() == null
113-
? SecurityUtility.createNettySslContextForClient(
114-
sslProvider,
115-
conf.isTlsAllowInsecureConnection(),
116-
conf.getTlsTrustCertsFilePath(),
117-
authData.getTlsCertificates(),
118-
authData.getTlsPrivateKey(),
119-
conf.getTlsCiphers(),
120-
conf.getTlsProtocols())
121-
: SecurityUtility.createNettySslContextForClient(sslProvider,
122-
conf.isTlsAllowInsecureConnection(),
123-
authData.getTlsTrustStoreStream(),
124-
authData.getTlsCertificates(), authData.getTlsPrivateKey(),
125-
conf.getTlsCiphers(),
126-
conf.getTlsProtocols());
127-
} else {
128-
return SecurityUtility.createNettySslContextForClient(
129-
sslProvider,
130-
conf.isTlsAllowInsecureConnection(),
131-
conf.getTlsTrustCertsFilePath(),
132-
conf.getTlsCertificateFilePath(),
133-
conf.getTlsKeyFilePath(),
134-
conf.getTlsCiphers(),
135-
conf.getTlsProtocols());
136-
}
137-
} catch (Exception e) {
138-
throw new RuntimeException("Failed to create TLS context", e);
139-
}
140-
}, TLS_CERTIFICATE_CACHE_MILLIS, TimeUnit.MILLISECONDS);
141-
} else {
142-
sslContextSupplier = null;
120+
authData1));
143121
}
122+
throw new PulsarClientException(
123+
"Failed to create TLS context, the tlsEnabledWithKeyStore need to be true");
144124
}
145125

146-
@Override
147-
public void initChannel(SocketChannel ch) throws Exception {
148-
ch.pipeline().addLast("consolidation", new FlushConsolidationHandler(1024, true));
149-
150-
// Setup channel except for the SsHandler for TLS enabled connections
151-
ch.pipeline().addLast("ByteBufPairEncoder", ByteBufPair.getEncoder(tlsEnabled));
126+
private Supplier<SslContext> getSslContextSupplier(String host) {
127+
return sslContextSupplierMap.computeIfAbsent(host, key -> new ObjectCache<>(() -> {
128+
try {
129+
SslProvider sslProvider = null;
130+
if (conf.getSslProvider() != null) {
131+
sslProvider = SslProvider.valueOf(conf.getSslProvider());
132+
}
152133

153-
ch.pipeline().addLast("frameDecoder", new LengthFieldBasedFrameDecoder(
154-
Commands.DEFAULT_MAX_MESSAGE_SIZE + Commands.MESSAGE_SIZE_FRAME_PADDING, 0, 4, 0, 4));
155-
ChannelHandler clientCnx = clientCnxSupplier.get();
156-
ch.pipeline().addLast("handler", clientCnx);
134+
// Set client certificate if available
135+
AuthenticationDataProvider authData = conf.getAuthentication().getAuthData(host);
136+
if (authData.hasDataForTls()) {
137+
return authData.getTlsTrustStoreStream() == null
138+
? SecurityUtility.createNettySslContextForClient(
139+
sslProvider,
140+
conf.isTlsAllowInsecureConnection(),
141+
conf.getTlsTrustCertsFilePath(),
142+
authData.getTlsCertificates(),
143+
authData.getTlsPrivateKey(),
144+
conf.getTlsCiphers(),
145+
conf.getTlsProtocols())
146+
: SecurityUtility.createNettySslContextForClient(sslProvider,
147+
conf.isTlsAllowInsecureConnection(),
148+
authData.getTlsTrustStoreStream(),
149+
authData.getTlsCertificates(), authData.getTlsPrivateKey(),
150+
conf.getTlsCiphers(),
151+
conf.getTlsProtocols());
152+
} else {
153+
return SecurityUtility.createNettySslContextForClient(
154+
sslProvider,
155+
conf.isTlsAllowInsecureConnection(),
156+
conf.getTlsTrustCertsFilePath(),
157+
conf.getTlsCertificateFilePath(),
158+
conf.getTlsKeyFilePath(),
159+
conf.getTlsCiphers(),
160+
conf.getTlsProtocols());
161+
}
162+
} catch (Exception e) {
163+
throw new RuntimeException("Failed to create TLS context", e);
164+
}
165+
}, TLS_CERTIFICATE_CACHE_MILLIS, TimeUnit.MILLISECONDS));
157166
}
158167

159168
/**
@@ -175,9 +184,10 @@ CompletableFuture<Channel> initTls(Channel ch, InetSocketAddress sniHost) {
175184
ch.eventLoop().execute(() -> {
176185
try {
177186
SslHandler handler = tlsEnabledWithKeyStore
178-
? new SslHandler(nettySSLContextAutoRefreshBuilder.get()
179-
.createSSLEngine(sniHost.getHostString(), sniHost.getPort()))
180-
: sslContextSupplier.get().newHandler(ch.alloc(), sniHost.getHostString(), sniHost.getPort());
187+
? new SslHandler(getNettySSLContextAutoRefreshBuilder(sniHost.getHostName()).get()
188+
.createSSLEngine(sniHost.getHostString(), sniHost.getPort()))
189+
: getSslContextSupplier(sniHost.getHostName()).get()
190+
.newHandler(ch.alloc(), sniHost.getHostString(), sniHost.getPort());
181191

182192
if (tlsHostnameVerificationEnabled) {
183193
SecurityUtility.configureSSLHandler(handler);

0 commit comments

Comments
 (0)