提交 577b8976 编写于 作者: M Matteo Merli 提交者: Rajan Dhabalia

Ensure producer/consumer are creating before upgrading from HTTP to WebSocket (#779)

* Ensure producer/consumer are creating before upgrading from HTTP to WebSocket

* Addressed PR comments
上级 696c4a9d
......@@ -35,7 +35,7 @@ import javax.net.ssl.SSLContext;
import javax.net.ssl.TrustManager;
import org.apache.bookkeeper.test.PortManager;
import org.apache.pulsar.client.api.ProducerConsumerBase;
import org.apache.pulsar.client.api.TlsProducerConsumerTest;
import org.apache.pulsar.websocket.WebSocketService;
import org.apache.pulsar.websocket.service.ProxyServer;
import org.apache.pulsar.websocket.service.WebSocketProxyConfiguration;
......@@ -53,7 +53,7 @@ import org.testng.annotations.Test;
import io.netty.handler.ssl.util.InsecureTrustManagerFactory;
public class ProxyPublishConsumeTls extends ProducerConsumerBase {
public class ProxyPublishConsumeTls extends TlsProducerConsumerTest {
protected String methodName;
private int port;
private int tlsPort;
......@@ -65,8 +65,9 @@ public class ProxyPublishConsumeTls extends ProducerConsumerBase {
@BeforeMethod
public void setup() throws Exception {
super.internalSetup();
super.producerBaseSetup();
super.setup();
this.internalSetupForTls();
port = PortManager.nextFreePort();
tlsPort = PortManager.nextFreePort();
......@@ -87,7 +88,7 @@ public class ProxyPublishConsumeTls extends ProducerConsumerBase {
@AfterMethod
protected void cleanup() throws Exception {
super.internalCleanup();
super.cleanup();
service.close();
proxyServer.stop();
log.info("Finished Cleaning Up Test setup");
......@@ -112,7 +113,6 @@ public class ProxyPublishConsumeTls extends ProducerConsumerBase {
WebSocketClient consumeClient = new WebSocketClient(sslContextFactory);
SimpleConsumerSocket consumeSocket = new SimpleConsumerSocket();
WebSocketClient produceClient = new WebSocketClient(sslContextFactory);
SimpleProducerSocket produceSocket = new SimpleProducerSocket();
try {
consumeClient.start();
......@@ -121,8 +121,7 @@ public class ProxyPublishConsumeTls extends ProducerConsumerBase {
log.info("Connecting to : {}", consumeUri);
Assert.assertTrue(consumerFuture.get().isOpen());
Thread.sleep(500);
SimpleProducerSocket produceSocket = new SimpleProducerSocket();
ClientUpgradeRequest produceRequest = new ClientUpgradeRequest();
produceClient.start();
Future<Session> producerFuture = produceClient.connect(produceSocket, produceUri, produceRequest);
......
......@@ -25,14 +25,15 @@ import java.io.IOException;
import java.util.List;
import java.util.Map;
import java.util.TreeMap;
import java.util.concurrent.CompletableFuture;
import javax.naming.AuthenticationException;
import javax.servlet.http.HttpServletRequest;
import javax.servlet.http.HttpServletResponse;
import org.apache.pulsar.common.naming.DestinationName;
import org.eclipse.jetty.websocket.api.Session;
import org.eclipse.jetty.websocket.api.WebSocketAdapter;
import org.eclipse.jetty.websocket.servlet.ServletUpgradeResponse;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
......@@ -46,7 +47,7 @@ public abstract class AbstractWebSocketHandler extends WebSocketAdapter implemen
protected final String topic;
protected final Map<String, String> queryParams;
public AbstractWebSocketHandler(WebSocketService service, HttpServletRequest request) {
public AbstractWebSocketHandler(WebSocketService service, HttpServletRequest request, ServletUpgradeResponse response) {
this.service = service;
this.request = request;
this.topic = extractTopicName(request);
......@@ -55,24 +56,27 @@ public abstract class AbstractWebSocketHandler extends WebSocketAdapter implemen
request.getParameterMap().forEach((key, values) -> {
queryParams.put(key, values[0]);
});
}
@Override
public void onWebSocketConnect(Session session) {
super.onWebSocketConnect(session);
log.info("[{}] New WebSocket session on topic {}", session.getRemoteAddress(), topic);
checkAuth(response);
}
private void checkAuth(ServletUpgradeResponse response) {
String authRole = "<none>";
if (service.isAuthenticationEnabled()) {
try {
authRole = service.getAuthenticationService().authenticateHttpRequest(request);
log.info("[{}] Authenticated WebSocket client {} on topic {}", session.getRemoteAddress(), authRole,
topic);
log.info("[{}:{}] Authenticated WebSocket client {} on topic {}", request.getRemoteAddr(),
request.getRemotePort(), authRole, topic);
} catch (AuthenticationException e) {
log.warn("[{}] Failed to authenticated WebSocket client {} on topic {}: {}",
session.getRemoteAddress(), authRole, topic, e.getMessage());
close(WebSocketError.AuthenticationError);
log.warn("[{}:{}] Failed to authenticated WebSocket client {} on topic {}: {}", request.getRemoteAddr(),
request.getRemotePort(), authRole, topic, e.getMessage());
try {
response.sendError(HttpServletResponse.SC_UNAUTHORIZED, "Failed to authenticate");
} catch (IOException e1) {
log.warn("[{}:{}] Failed to send error: {}", request.getRemoteAddr(), request.getRemotePort(),
e1.getMessage(), e1);
}
return;
}
}
......@@ -80,19 +84,30 @@ public abstract class AbstractWebSocketHandler extends WebSocketAdapter implemen
if (service.isAuthorizationEnabled()) {
try {
if (!isAuthorized(authRole)) {
log.warn("[{}] WebSocket Client [{}] is not authorized on topic {}", session.getRemoteAddress(), authRole,
topic);
close(WebSocketError.NotAuthorizedError);
log.warn("[{}:{}] WebSocket Client [{}] is not authorized on topic {}", request.getRemoteAddr(),
request.getRemotePort(), authRole, topic);
response.sendError(HttpServletResponse.SC_FORBIDDEN, "Not authorized");
return;
}
} catch (Exception e) {
log.warn("[{}] Got an exception when authorizing WebSocket client {} on topic {} on: {}",
session.getRemoteAddress(), authRole, topic, e.getMessage());
close(WebSocketError.UnknownError);
log.warn("[{}:{}] Got an exception when authorizing WebSocket client {} on topic {} on: {}",
request.getRemoteAddr(), request.getRemotePort(), authRole, topic, e.getMessage());
try {
response.sendError(HttpServletResponse.SC_INTERNAL_SERVER_ERROR, "Server error");
} catch (IOException e1) {
log.warn("[{}:{}] Failed to send error: {}", request.getRemoteAddr(), request.getRemotePort(),
e1.getMessage(), e1);
}
return;
}
}
createClient(session);
}
@Override
public void onWebSocketConnect(Session session) {
super.onWebSocketConnect(session);
log.info("[{}] New WebSocket session on topic {}", session.getRemoteAddress(), topic);
}
@Override
......@@ -153,7 +168,5 @@ public abstract class AbstractWebSocketHandler extends WebSocketAdapter implemen
protected abstract Boolean isAuthorized(String authRole) throws Exception;
protected abstract void createClient(Session session);
private static final Logger log = LoggerFactory.getLogger(AbstractWebSocketHandler.class);
}
......@@ -26,13 +26,13 @@ import java.time.ZoneId;
import java.time.format.DateTimeFormatter;
import java.util.Base64;
import java.util.List;
import java.util.concurrent.CompletableFuture;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicLongFieldUpdater;
import java.util.concurrent.atomic.LongAdder;
import javax.servlet.http.HttpServletRequest;
import javax.servlet.http.HttpServletResponse;
import org.apache.pulsar.client.api.Consumer;
import org.apache.pulsar.client.api.ConsumerConfiguration;
......@@ -42,8 +42,8 @@ import org.apache.pulsar.common.naming.DestinationName;
import org.apache.pulsar.common.util.ObjectMapperFactory;
import org.apache.pulsar.websocket.data.ConsumerAck;
import org.apache.pulsar.websocket.data.ConsumerMessage;
import org.eclipse.jetty.websocket.api.Session;
import org.eclipse.jetty.websocket.api.WriteCallback;
import org.eclipse.jetty.websocket.servlet.ServletUpgradeResponse;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
......@@ -69,7 +69,7 @@ public class ConsumerHandler extends AbstractWebSocketHandler {
private final int maxPendingMessages;
private final AtomicInteger pendingMessages = new AtomicInteger();
private final LongAdder numMsgsDelivered;
private final LongAdder numBytesDelivered;
private final LongAdder numMsgsAcked;
......@@ -77,27 +77,28 @@ public class ConsumerHandler extends AbstractWebSocketHandler {
private static final AtomicLongFieldUpdater<ConsumerHandler> MSG_DELIVERED_COUNTER_UPDATER =
AtomicLongFieldUpdater.newUpdater(ConsumerHandler.class, "msgDeliveredCounter");
public ConsumerHandler(WebSocketService service, HttpServletRequest request) {
super(service, request);
public ConsumerHandler(WebSocketService service, HttpServletRequest request, ServletUpgradeResponse response) {
super(service, request, response);
this.subscription = extractSubscription(request);
this.conf = getConsumerConfiguration();
this.maxPendingMessages = (conf.getReceiverQueueSize() == 0) ? 1 : conf.getReceiverQueueSize();
this.numMsgsDelivered = new LongAdder();
this.numBytesDelivered = new LongAdder();
this.numMsgsAcked = new LongAdder();
}
@Override
protected void createClient(Session session) {
try {
this.consumer = service.getPulsarClient().subscribe(topic, subscription, conf);
this.service.addConsumer(this);
receiveMessage();
} catch (Exception e) {
log.warn("[{}] Failed in creating subscription {} on topic {}", session.getRemoteAddress(), subscription,
topic, e);
close(WebSocketError.FailedToSubscribe, e.getMessage());
log.warn("[{}:{}] Failed in creating subscription {} on topic {}", request.getRemoteAddr(),
request.getRemotePort(), subscription, topic, e);
try {
response.sendError(HttpServletResponse.SC_INTERNAL_SERVER_ERROR, "Failed to subscribe");
} catch (IOException e1) {
log.warn("[{}:{}] Failed to send error: {}", request.getRemoteAddr(), request.getRemotePort(),
e1.getMessage(), e1);
}
}
}
......@@ -224,7 +225,7 @@ public class ConsumerHandler extends AbstractWebSocketHandler {
public long getMsgDeliveredCounter() {
return MSG_DELIVERED_COUNTER_UPDATER.get(this);
}
protected void updateDeliverMsgStat(long msgSize) {
numMsgsDelivered.increment();
MSG_DELIVERED_COUNTER_UPDATER.incrementAndGet(this);
......
......@@ -32,6 +32,7 @@ import java.util.concurrent.atomic.AtomicLongFieldUpdater;
import java.util.concurrent.atomic.LongAdder;
import javax.servlet.http.HttpServletRequest;
import javax.servlet.http.HttpServletResponse;
import org.apache.pulsar.client.api.CompressionType;
import org.apache.pulsar.client.api.Message;
......@@ -45,6 +46,8 @@ import org.apache.pulsar.websocket.data.ProducerAck;
import org.apache.pulsar.websocket.data.ProducerMessage;
import org.apache.pulsar.websocket.stats.StatsBuckets;
import org.eclipse.jetty.websocket.api.Session;
import org.eclipse.jetty.websocket.api.UpgradeResponse;
import org.eclipse.jetty.websocket.servlet.ServletUpgradeResponse;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
......@@ -70,16 +73,31 @@ public class ProducerHandler extends AbstractWebSocketHandler {
private volatile long msgPublishedCounter = 0;
private static final AtomicLongFieldUpdater<ProducerHandler> MSG_PUBLISHED_COUNTER_UPDATER =
AtomicLongFieldUpdater.newUpdater(ProducerHandler.class, "msgPublishedCounter");
public static final long[] ENTRY_LATENCY_BUCKETS_USEC = { 500, 1_000, 5_000, 10_000, 20_000, 50_000, 100_000,
200_000, 1000_000 };
public ProducerHandler(WebSocketService service, HttpServletRequest request) {
super(service, request);
public ProducerHandler(WebSocketService service, HttpServletRequest request, ServletUpgradeResponse response) {
super(service, request, response);
this.numMsgsSent = new LongAdder();
this.numBytesSent = new LongAdder();
this.numMsgsFailed = new LongAdder();
this.publishLatencyStatsUSec = new StatsBuckets(ENTRY_LATENCY_BUCKETS_USEC);
try {
ProducerConfiguration conf = getProducerConfiguration();
this.producer = service.getPulsarClient().createProducer(topic, conf);
this.service.addProducer(this);
} catch (Exception e) {
log.warn("[{}:{}] Failed in creating producer on topic {}", request.getRemoteAddr(),
request.getRemotePort(), topic, e);
try {
response.sendError(HttpServletResponse.SC_INTERNAL_SERVER_ERROR, "Failed to create producer");
} catch (IOException e1) {
log.warn("[{}:{}] Failed to send error: {}", request.getRemoteAddr(), request.getRemotePort(),
e1.getMessage(), e1);
}
}
}
@Override
......@@ -97,19 +115,6 @@ public class ProducerHandler extends AbstractWebSocketHandler {
}
}
@Override
protected void createClient(Session session) {
try {
ProducerConfiguration conf = getProducerConfiguration();
this.producer = service.getPulsarClient().createProducer(topic, conf);
this.service.addProducer(this);
} catch (Exception e) {
log.warn("[{}] Failed in creating producer on topic {}", session.getRemoteAddress(),
topic, e);
close(FailedToCreateProducer, e.getMessage());
}
}
@Override
public void onWebSocketText(String message) {
ProducerMessage sendRequest;
......@@ -201,7 +206,7 @@ public class ProducerHandler extends AbstractWebSocketHandler {
log.warn("[{}] Failed to send ack {}", producer.getTopic(), e.getMessage(), e);
}
}
private void updateSentMsgStats(long msgSize, long latencyUsec) {
this.publishLatencyStatsUSec.addValue(latencyUsec);
this.numBytesSent.add(msgSize);
......@@ -214,7 +219,7 @@ public class ProducerHandler extends AbstractWebSocketHandler {
// Set to false to prevent the server thread from being blocked if a lot of messages are pending.
conf.setBlockIfQueueFull(false);
if (queryParams.containsKey("sendTimeoutMillis")) {
conf.setSendTimeout(Integer.parseInt(queryParams.get("sendTimeoutMillis")), TimeUnit.MILLISECONDS);
}
......
......@@ -18,39 +18,36 @@
*/
package org.apache.pulsar.websocket;
import static com.google.common.base.Preconditions.checkArgument;
import static org.apache.commons.lang3.StringUtils.isNotBlank;
import java.io.IOException;
import java.time.Instant;
import java.time.ZoneId;
import java.time.format.DateTimeFormatter;
import java.util.Base64;
import java.util.List;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicLongFieldUpdater;
import java.util.concurrent.atomic.LongAdder;
import javax.servlet.http.HttpServletRequest;
import javax.servlet.http.HttpServletResponse;
import static org.apache.commons.lang3.StringUtils.isNotBlank;
import org.apache.pulsar.client.api.Consumer;
import org.apache.pulsar.client.api.MessageId;
import org.apache.pulsar.client.api.Reader;
import org.apache.pulsar.client.api.ReaderConfiguration;
import org.apache.pulsar.client.api.SubscriptionType;
import org.apache.pulsar.client.impl.MessageIdImpl;
import org.apache.pulsar.client.impl.ReaderImpl;
import org.apache.pulsar.client.api.Consumer;
import org.apache.pulsar.client.api.MessageId;
import org.apache.pulsar.client.api.Reader;
import org.apache.pulsar.common.naming.DestinationName;
import org.apache.pulsar.common.util.ObjectMapperFactory;
import org.apache.pulsar.websocket.data.ConsumerMessage;
import org.eclipse.jetty.websocket.api.Session;
import org.eclipse.jetty.websocket.api.WriteCallback;
import org.eclipse.jetty.websocket.servlet.ServletUpgradeResponse;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import com.fasterxml.jackson.core.JsonProcessingException;
import com.google.common.base.Splitter;
/**
*
......@@ -67,34 +64,35 @@ public class ReaderHandler extends AbstractWebSocketHandler {
private final int maxPendingMessages;
private final AtomicInteger pendingMessages = new AtomicInteger();
private final LongAdder numMsgsDelivered;
private final LongAdder numBytesDelivered;
private volatile long msgDeliveredCounter = 0;
private static final AtomicLongFieldUpdater<ReaderHandler> MSG_DELIVERED_COUNTER_UPDATER =
AtomicLongFieldUpdater.newUpdater(ReaderHandler.class, "msgDeliveredCounter");
public ReaderHandler(WebSocketService service, HttpServletRequest request) {
super(service, request);
public ReaderHandler(WebSocketService service, HttpServletRequest request, ServletUpgradeResponse response) {
super(service, request, response);
this.subscription = "";
this.conf = getReaderConfiguration();
this.maxPendingMessages = (conf.getReceiverQueueSize() == 0) ? 1 : conf.getReceiverQueueSize();
this.numMsgsDelivered = new LongAdder();
this.numBytesDelivered = new LongAdder();
}
@Override
protected void createClient(Session session) {
this.numBytesDelivered = new LongAdder();
try {
this.reader = service.getPulsarClient().createReader(topic, getMessageId(), conf);
this.subscription = ((ReaderImpl)this.reader).getConsumer().getSubscription();
this.subscription = ((ReaderImpl)this.reader).getConsumer().getSubscription();
this.service.addReader(this);
receiveMessage();
} catch (Exception e) {
log.warn("[{}] Failed in creating subscription {} on topic {}", session.getRemoteAddress(), subscription,
topic, e);
close(WebSocketError.FailedToSubscribe, e.getMessage());
log.warn("[{}:{}] Failed in creating reader {} on topic {}", request.getRemoteAddr(),
request.getRemotePort(), subscription, topic, e);
try {
response.sendError(HttpServletResponse.SC_INTERNAL_SERVER_ERROR, "Failed to create reader");
} catch (IOException e1) {
log.warn("[{}:{}] Failed to send error: {}", request.getRemoteAddr(), request.getRemotePort(),
e1.getMessage(), e1);
}
}
}
......@@ -184,7 +182,7 @@ public class ReaderHandler extends AbstractWebSocketHandler {
});
}
}
public Consumer getConsumer() {
return ((ReaderImpl)reader).getConsumer();
}
......@@ -192,7 +190,7 @@ public class ReaderHandler extends AbstractWebSocketHandler {
public String getSubscription() {
return subscription;
}
public SubscriptionType getSubscriptionType() {
return SubscriptionType.Exclusive;
}
......@@ -208,7 +206,7 @@ public class ReaderHandler extends AbstractWebSocketHandler {
public long getMsgDeliveredCounter() {
return MSG_DELIVERED_COUNTER_UPDATER.get(this);
}
protected void updateDeliverMsgStat(long msgSize) {
numMsgsDelivered.increment();
MSG_DELIVERED_COUNTER_UPDATER.incrementAndGet(this);
......
......@@ -37,6 +37,7 @@ public class WebSocketConsumerServlet extends WebSocketServlet {
public void configure(WebSocketServletFactory factory) {
factory.getPolicy().setMaxTextMessageSize(WebSocketService.MaxTextFrameSize);
factory.setCreator((request, response) -> new ConsumerHandler(service, request.getHttpServletRequest()));
factory.setCreator(
(request, response) -> new ConsumerHandler(service, request.getHttpServletRequest(), response));
}
}
\ No newline at end of file
......@@ -35,6 +35,6 @@ public class WebSocketProducerServlet extends WebSocketServlet {
@Override
public void configure(WebSocketServletFactory factory) {
factory.getPolicy().setMaxTextMessageSize(WebSocketService.MaxTextFrameSize);
factory.setCreator((req, resp) -> new ProducerHandler(service, req.getHttpServletRequest()));
factory.setCreator((request, response) -> new ProducerHandler(service, request.getHttpServletRequest(), response));
}
}
\ No newline at end of file
......@@ -37,6 +37,7 @@ public class WebSocketReaderServlet extends WebSocketServlet {
public void configure(WebSocketServletFactory factory) {
factory.getPolicy().setMaxTextMessageSize(WebSocketService.MaxTextFrameSize);
factory.setCreator((request, response) -> new ReaderHandler(service, request.getHttpServletRequest()));
factory.setCreator(
(request, response) -> new ReaderHandler(service, request.getHttpServletRequest(), response));
}
}
\ No newline at end of file
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册