Skip to content

Commit

Permalink
[ISSUE #4731] HttpRequestProcessor enhancement (#4732)
Browse files Browse the repository at this point in the history
* resolve conflicts.

* resolve
  • Loading branch information
karsonto authored Feb 18, 2024
1 parent d6393ab commit 9a3912a
Show file tree
Hide file tree
Showing 27 changed files with 295 additions and 151 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -27,7 +27,6 @@
import org.apache.eventmesh.common.protocol.http.common.RequestCode;
import org.apache.eventmesh.common.protocol.http.header.Header;
import org.apache.eventmesh.common.utils.AssertUtils;
import org.apache.eventmesh.runtime.common.Pair;
import org.apache.eventmesh.runtime.configuration.EventMeshHTTPConfiguration;
import org.apache.eventmesh.runtime.constants.EventMeshConstants;
import org.apache.eventmesh.runtime.core.protocol.http.async.AsyncContext;
Expand Down Expand Up @@ -117,7 +116,7 @@ public abstract class AbstractHTTPServer extends AbstractRemotingServer {
/**
* key: request code
*/
protected final transient Map<String, Pair<HttpRequestProcessor, ThreadPoolExecutor>> httpRequestProcessorTable =
protected final transient Map<String, HttpRequestProcessor> httpRequestProcessorTable =
new ConcurrentHashMap<>(64);

private HttpConnectionHandler httpConnectionHandler;
Expand Down Expand Up @@ -198,11 +197,10 @@ public void shutdown() throws Exception {
/**
* Registers the processors required by the runtime module
*/
public void registerProcessor(final Integer requestCode, final HttpRequestProcessor processor, final ThreadPoolExecutor executor) {
public void registerProcessor(final Integer requestCode, final HttpRequestProcessor processor) {
AssertUtils.notNull(requestCode, "requestCode can't be null");
AssertUtils.notNull(processor, "processor can't be null");
AssertUtils.notNull(executor, "executor can't be null");
this.httpRequestProcessorTable.put(requestCode.toString(), new Pair<>(processor, executor));
this.httpRequestProcessorTable.putIfAbsent(requestCode.toString(), processor);
}

/**
Expand Down Expand Up @@ -400,46 +398,54 @@ private void injectHttpRequestHeader(final ChannelHandlerContext ctx, final Http

private void processHttpCommandRequest(final ChannelHandlerContext ctx, final AsyncContext<HttpCommand> asyncContext) {
final HttpCommand request = asyncContext.getRequest();
final Pair<HttpRequestProcessor, ThreadPoolExecutor> choosed = httpRequestProcessorTable.get(request.getRequestCode());
try {
choosed.getObject2().submit(() -> {
try {
final HttpRequestProcessor processor = choosed.getObject1();
if (processor.rejectRequest()) {
final HttpCommand responseCommand =
request.createHttpCommandResponse(EventMeshRetCode.EVENTMESH_REJECT_BY_PROCESSOR_ERROR);
asyncContext.onComplete(responseCommand);

if (asyncContext.isComplete()) {
sendResponse(ctx, responseCommand.httpResponse());
log.debug("{}", asyncContext.getResponse());
final Map<String, Object> traceMap = asyncContext.getRequest().getHeader().toMap();
TraceUtils.finishSpanWithException(TraceUtils.prepareServerSpan(traceMap,
EventMeshTraceConstants.TRACE_UPSTREAM_EVENTMESH_SERVER_SPAN,
false),
traceMap,
EventMeshRetCode.EVENTMESH_REJECT_BY_PROCESSOR_ERROR.getErrMsg(), null);
}

return;
final HttpRequestProcessor choosed = httpRequestProcessorTable.get(request.getRequestCode());
Runnable runnable = () -> {
try {
final HttpRequestProcessor processor = choosed;
if (processor.rejectRequest()) {
final HttpCommand responseCommand =
request.createHttpCommandResponse(EventMeshRetCode.EVENTMESH_REJECT_BY_PROCESSOR_ERROR);
asyncContext.onComplete(responseCommand);

if (asyncContext.isComplete()) {
sendResponse(ctx, responseCommand.httpResponse());
log.debug("{}", asyncContext.getResponse());
final Map<String, Object> traceMap = asyncContext.getRequest().getHeader().toMap();
TraceUtils.finishSpanWithException(TraceUtils.prepareServerSpan(traceMap,

EventMeshTraceConstants.TRACE_UPSTREAM_EVENTMESH_SERVER_SPAN,
false),
traceMap,
EventMeshRetCode.EVENTMESH_REJECT_BY_PROCESSOR_ERROR.getErrMsg(), null);
}

processor.processRequest(ctx, asyncContext);
if (!asyncContext.isComplete()) {
return;
}
return;
}

processor.processRequest(ctx, asyncContext);
if (!asyncContext.isComplete()) {
return;
}

metrics.getSummaryMetrics()
.recordHTTPReqResTimeCost(System.currentTimeMillis() - request.getReqTime());
log.debug("{}", asyncContext.getResponse());
metrics.getSummaryMetrics()
.recordHTTPReqResTimeCost(System.currentTimeMillis() - request.getReqTime());
sendResponse(ctx, asyncContext.getResponse().httpResponse());

log.debug("{}", asyncContext.getResponse());
} catch (Exception e) {
log.error("process error", e);
}
};

sendResponse(ctx, asyncContext.getResponse().httpResponse());
try {
if (Objects.nonNull(choosed.executor())) {
choosed.executor().execute(() -> {
runnable.run();
});
} else {
runnable.run();
}

} catch (Exception e) {
log.error("process error", e);
}
});
} catch (RejectedExecutionException re) {
asyncContext.onComplete(request.createHttpCommandResponse(EventMeshRetCode.OVERLOAD));
metrics.getSummaryMetrics().recordHTTPDiscard();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -62,7 +62,6 @@

import java.util.List;
import java.util.Optional;
import java.util.concurrent.ThreadPoolExecutor;

import org.assertj.core.util.Lists;

Expand All @@ -71,6 +70,7 @@

import lombok.extern.slf4j.Slf4j;


/**
* Add multiple managers to the underlying server
*/
Expand All @@ -84,22 +84,16 @@ public class EventMeshHTTPServer extends AbstractHTTPServer {

private final Acl acl;
private final EventBus eventBus = new EventBus();

private final transient HTTPClientPool httpClientPool = new HTTPClientPool(10);
private ConsumerManager consumerManager;
private ProducerManager producerManager;
private SubscriptionManager subscriptionManager;

private FilterEngine filterEngine;

private TransformerEngine transformerEngine;

private HttpRetryer httpRetryer;

private transient RateLimiter msgRateLimiter;
private transient RateLimiter batchRateLimiter;

private final transient HTTPClientPool httpClientPool = new HTTPClientPool(10);

public EventMeshHTTPServer(final EventMeshServer eventMeshServer, final EventMeshHTTPConfiguration eventMeshHttpConfiguration) {

super(eventMeshHttpConfiguration.getHttpServerPort(),
Expand Down Expand Up @@ -239,68 +233,59 @@ private void unRegister() {
}

private void registerHTTPRequestProcessor() throws Exception {
HTTPThreadPoolGroup httpThreadPoolGroup = super.getHttpThreadPoolGroup();

ThreadPoolExecutor batchMsgExecutor = httpThreadPoolGroup.getBatchMsgExecutor();
final BatchSendMessageProcessor batchSendMessageProcessor = new BatchSendMessageProcessor(this);
registerProcessor(RequestCode.MSG_BATCH_SEND.getRequestCode(), batchSendMessageProcessor, batchMsgExecutor);
registerProcessor(RequestCode.MSG_BATCH_SEND.getRequestCode(), batchSendMessageProcessor);

final BatchSendMessageV2Processor batchSendMessageV2Processor = new BatchSendMessageV2Processor(this);
registerProcessor(RequestCode.MSG_BATCH_SEND_V2.getRequestCode(), batchSendMessageV2Processor,
batchMsgExecutor);
registerProcessor(RequestCode.MSG_BATCH_SEND_V2.getRequestCode(), batchSendMessageV2Processor);

ThreadPoolExecutor sendMsgExecutor = httpThreadPoolGroup.getSendMsgExecutor();
final SendSyncMessageProcessor sendSyncMessageProcessor = new SendSyncMessageProcessor(this);
registerProcessor(RequestCode.MSG_SEND_SYNC.getRequestCode(), sendSyncMessageProcessor, sendMsgExecutor);
registerProcessor(RequestCode.MSG_SEND_SYNC.getRequestCode(), sendSyncMessageProcessor);

final SendAsyncMessageProcessor sendAsyncMessageProcessor = new SendAsyncMessageProcessor(this);
registerProcessor(RequestCode.MSG_SEND_ASYNC.getRequestCode(), sendAsyncMessageProcessor, sendMsgExecutor);
registerProcessor(RequestCode.MSG_SEND_ASYNC.getRequestCode(), sendAsyncMessageProcessor);

final SendAsyncEventProcessor sendAsyncEventProcessor = new SendAsyncEventProcessor(this);
this.getHandlerService().register(sendAsyncEventProcessor, sendMsgExecutor);
this.getHandlerService().register(sendAsyncEventProcessor);

ThreadPoolExecutor remoteMsgExecutor = httpThreadPoolGroup.getRemoteMsgExecutor();
final SendAsyncRemoteEventProcessor sendAsyncRemoteEventProcessor = new SendAsyncRemoteEventProcessor(this);
this.getHandlerService().register(sendAsyncRemoteEventProcessor, remoteMsgExecutor);
this.getHandlerService().register(sendAsyncRemoteEventProcessor);

ThreadPoolExecutor runtimeAdminExecutor = httpThreadPoolGroup.getRuntimeAdminExecutor();
final AdminMetricsProcessor adminMetricsProcessor = new AdminMetricsProcessor(this);
registerProcessor(RequestCode.ADMIN_METRICS.getRequestCode(), adminMetricsProcessor, runtimeAdminExecutor);
registerProcessor(RequestCode.ADMIN_METRICS.getRequestCode(), adminMetricsProcessor);

ThreadPoolExecutor clientManageExecutor = httpThreadPoolGroup.getClientManageExecutor();
final HeartBeatProcessor heartProcessor = new HeartBeatProcessor(this);
registerProcessor(RequestCode.HEARTBEAT.getRequestCode(), heartProcessor, clientManageExecutor);
registerProcessor(RequestCode.HEARTBEAT.getRequestCode(), heartProcessor);

final SubscribeProcessor subscribeProcessor = new SubscribeProcessor(this);
registerProcessor(RequestCode.SUBSCRIBE.getRequestCode(), subscribeProcessor, clientManageExecutor);
registerProcessor(RequestCode.SUBSCRIBE.getRequestCode(), subscribeProcessor);

final LocalSubscribeEventProcessor localSubscribeEventProcessor = new LocalSubscribeEventProcessor(this);
this.getHandlerService().register(localSubscribeEventProcessor, clientManageExecutor);
this.getHandlerService().register(localSubscribeEventProcessor);

final RemoteSubscribeEventProcessor remoteSubscribeEventProcessor = new RemoteSubscribeEventProcessor(this);
this.getHandlerService().register(remoteSubscribeEventProcessor, clientManageExecutor);
this.getHandlerService().register(remoteSubscribeEventProcessor);

final UnSubscribeProcessor unSubscribeProcessor = new UnSubscribeProcessor(this);
registerProcessor(RequestCode.UNSUBSCRIBE.getRequestCode(), unSubscribeProcessor, clientManageExecutor);
registerProcessor(RequestCode.UNSUBSCRIBE.getRequestCode(), unSubscribeProcessor);

final LocalUnSubscribeEventProcessor localUnSubscribeEventProcessor = new LocalUnSubscribeEventProcessor(this);
this.getHandlerService().register(localUnSubscribeEventProcessor, clientManageExecutor);
this.getHandlerService().register(localUnSubscribeEventProcessor);

final RemoteUnSubscribeEventProcessor remoteUnSubscribeEventProcessor = new RemoteUnSubscribeEventProcessor(this);
this.getHandlerService().register(remoteUnSubscribeEventProcessor, clientManageExecutor);
this.getHandlerService().register(remoteUnSubscribeEventProcessor);

ThreadPoolExecutor replyMsgExecutor = httpThreadPoolGroup.getReplyMsgExecutor();
final ReplyMessageProcessor replyMessageProcessor = new ReplyMessageProcessor(this);
registerProcessor(RequestCode.REPLY_MESSAGE.getRequestCode(), replyMessageProcessor, replyMsgExecutor);
registerProcessor(RequestCode.REPLY_MESSAGE.getRequestCode(), replyMessageProcessor);

final CreateTopicProcessor createTopicProcessor = new CreateTopicProcessor(this);
this.getHandlerService().register(createTopicProcessor, clientManageExecutor);
this.getHandlerService().register(createTopicProcessor);

final DeleteTopicProcessor deleteTopicProcessor = new DeleteTopicProcessor(this);
this.getHandlerService().register(deleteTopicProcessor, clientManageExecutor);
this.getHandlerService().register(deleteTopicProcessor);

final QuerySubscriptionProcessor querySubscriptionProcessor = new QuerySubscriptionProcessor(this);
this.getHandlerService().register(querySubscriptionProcessor, clientManageExecutor);
this.getHandlerService().register(querySubscriptionProcessor);

registerWebhook();
}
Expand Down Expand Up @@ -370,4 +355,6 @@ public MetaStorage getMetaStorage() {
public HTTPClientPool getHttpClientPool() {
return httpClientPool;
}


}
Original file line number Diff line number Diff line change
@@ -0,0 +1,24 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You 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 org.apache.eventmesh.runtime.core.protocol.http.processor;

import org.apache.eventmesh.runtime.core.protocol.http.processor.inf.HttpRequestProcessor;

public abstract class AbstractHttpRequestProcessor implements HttpRequestProcessor {

}
Original file line number Diff line number Diff line change
Expand Up @@ -20,18 +20,25 @@
import org.apache.eventmesh.common.protocol.http.HttpCommand;
import org.apache.eventmesh.runtime.boot.EventMeshHTTPServer;
import org.apache.eventmesh.runtime.core.protocol.http.async.AsyncContext;
import org.apache.eventmesh.runtime.core.protocol.http.processor.inf.HttpRequestProcessor;

import java.util.concurrent.Executor;

import io.netty.channel.ChannelHandlerContext;

import lombok.RequiredArgsConstructor;


@RequiredArgsConstructor
public class AdminMetricsProcessor implements HttpRequestProcessor {
public class AdminMetricsProcessor extends AbstractHttpRequestProcessor {

private final EventMeshHTTPServer eventMeshHTTPServer;

@Override
public void processRequest(ChannelHandlerContext ctx, AsyncContext<HttpCommand> asyncContext) throws Exception {
}

@Override
public Executor executor() {
return eventMeshHTTPServer.getHttpThreadPoolGroup().getRuntimeAdminExecutor();
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -24,9 +24,10 @@
import org.apache.eventmesh.runtime.boot.EventMeshServer;
import org.apache.eventmesh.runtime.constants.EventMeshConstants;
import org.apache.eventmesh.runtime.core.protocol.http.async.AsyncContext;
import org.apache.eventmesh.runtime.core.protocol.http.processor.inf.HttpRequestProcessor;
import org.apache.eventmesh.runtime.util.RemotingHelper;

import java.util.concurrent.Executor;

import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand All @@ -35,7 +36,7 @@
import lombok.RequiredArgsConstructor;

@RequiredArgsConstructor
public class AdminShutdownProcessor implements HttpRequestProcessor {
public class AdminShutdownProcessor extends AbstractHttpRequestProcessor {

public final Logger cmdLogger = LoggerFactory.getLogger(EventMeshConstants.CMD);

Expand All @@ -54,4 +55,11 @@ public void processRequest(ChannelHandlerContext ctx, AsyncContext<HttpCommand>
HttpCommand responseEventMeshCommand = asyncContext.getRequest().createHttpCommandResponse(EventMeshRetCode.SUCCESS);
asyncContext.onComplete(responseEventMeshCommand);
}

@Override
public Executor executor() {
return (Runnable runnable) -> {
runnable.run();
};
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,6 @@
import org.apache.eventmesh.runtime.configuration.EventMeshHTTPConfiguration;
import org.apache.eventmesh.runtime.constants.EventMeshConstants;
import org.apache.eventmesh.runtime.core.protocol.http.async.AsyncContext;
import org.apache.eventmesh.runtime.core.protocol.http.processor.inf.HttpRequestProcessor;
import org.apache.eventmesh.runtime.core.protocol.producer.EventMeshProducer;
import org.apache.eventmesh.runtime.core.protocol.producer.SendMessageContext;
import org.apache.eventmesh.runtime.util.RemotingHelper;
Expand All @@ -53,6 +52,7 @@
import java.util.List;
import java.util.Map;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.Executor;
import java.util.concurrent.TimeUnit;

import org.slf4j.Logger;
Expand All @@ -65,7 +65,7 @@

import com.google.common.base.Stopwatch;

public class BatchSendMessageProcessor implements HttpRequestProcessor {
public class BatchSendMessageProcessor extends AbstractHttpRequestProcessor {

private static final Logger CMD_LOGGER = LoggerFactory.getLogger(EventMeshConstants.CMD);

Expand Down Expand Up @@ -288,4 +288,9 @@ public void onException(OnExceptionContext context) {
SendMessageBatchResponseBody.class);
return;
}

@Override
public Executor executor() {
return eventMeshHTTPServer.getHttpThreadPoolGroup().getBatchMsgExecutor();
}
}
Loading

0 comments on commit 9a3912a

Please sign in to comment.