[NO ISSUE][OTH]: Add methods to collect requests/jobs stats
Change-Id: Ib266878bf05782506045abfadaa83b41f0f95986
Reviewed-on: https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/17839
Integration-Tests: Jenkins <jenkins@fulliautomatix.ics.uci.edu>
Tested-by: Jenkins <jenkins@fulliautomatix.ics.uci.edu>
Reviewed-by: Hussain Towaileb <hussainht@gmail.com>
Reviewed-by: Peeyush Gupta <peeyush.gupta@couchbase.com>
diff --git a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/IActiveNotificationHandler.java b/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/IActiveNotificationHandler.java
index 60bfa69..9a05148 100644
--- a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/IActiveNotificationHandler.java
+++ b/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/IActiveNotificationHandler.java
@@ -19,6 +19,8 @@
*/
package org.apache.asterix.active;
+import java.util.Collection;
+
import org.apache.asterix.active.message.ActivePartitionMessage;
import org.apache.hyracks.api.exceptions.HyracksDataException;
@@ -53,6 +55,11 @@
void unregisterListener(IActiveEntityEventsListener listener) throws HyracksDataException;
/**
+ * @return List of all the registered event listeners
+ */
+ Collection<IActiveEntityEventsListener> getEventListenersAsList();
+
+ /**
* @return all the registered event listeners
*/
IActiveEntityEventsListener[] getEventListeners();
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/active/ActiveNotificationHandler.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/active/ActiveNotificationHandler.java
index 6b3581e..3c277d5 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/active/ActiveNotificationHandler.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/active/ActiveNotificationHandler.java
@@ -19,6 +19,7 @@
package org.apache.asterix.app.active;
import java.util.Collection;
+import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
@@ -187,6 +188,11 @@
}
@Override
+ public synchronized Collection<IActiveEntityEventsListener> getEventListenersAsList() {
+ return Collections.unmodifiableCollection(entityEventListeners.values());
+ }
+
+ @Override
public synchronized void registerListener(IActiveEntityEventsListener listener) throws HyracksDataException {
if (suspended) {
throw new RuntimeDataException(ErrorCode.ACTIVE_NOTIFICATION_HANDLER_IS_SUSPENDED);
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/api/IRequestTracker.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/api/IRequestTracker.java
index a3ddb30..0019015 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/api/IRequestTracker.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/api/IRequestTracker.java
@@ -75,4 +75,10 @@
* @return the recently completed requests
*/
Collection<IClientRequest> getCompletedRequests();
+
+ /**
+ *
+ * @return the total number of requests since cluster start/restart
+ */
+ long getTotalNumberOfRequests();
}
diff --git a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/utils/RequestTracker.java b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/utils/RequestTracker.java
index 333c709..c9425c6 100644
--- a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/utils/RequestTracker.java
+++ b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/utils/RequestTracker.java
@@ -24,6 +24,7 @@
import java.util.Map;
import java.util.Objects;
import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.atomic.AtomicLong;
import org.apache.asterix.common.api.IClientRequest;
import org.apache.asterix.common.api.IRequestTracker;
@@ -37,10 +38,12 @@
private final Map<String, IClientRequest> clientIdRequests = new ConcurrentHashMap<>();
private final CircularFifoQueue<IClientRequest> completedRequests;
private final ICcApplicationContext ccAppCtx;
+ private final AtomicLong numRequests;
public RequestTracker(ICcApplicationContext ccAppCtx) {
this.ccAppCtx = ccAppCtx;
completedRequests = new CircularFifoQueue<>(ccAppCtx.getExternalProperties().getRequestsArchiveSize());
+ numRequests = new AtomicLong(0);
}
@Override
@@ -57,6 +60,7 @@
@Override
public void track(IClientRequest request) {
runningRequests.put(request.getId(), request);
+ numRequests.incrementAndGet();
if (request.getClientContextId() != null) {
clientIdRequests.put(request.getClientContextId(), request);
}
@@ -112,4 +116,8 @@
private synchronized void archive(IClientRequest request) {
completedRequests.add(request);
}
+
+ public long getTotalNumberOfRequests() {
+ return numRequests.get();
+ }
}
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/IJobManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/IJobManager.java
index a9ddee3..b2795d4 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/IJobManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/IJobManager.java
@@ -112,6 +112,11 @@
Collection<JobRun> getRunningJobs();
/**
+ * @return number of jobs that are currently running.
+ */
+ int getRunningJobsCount();
+
+ /**
* @return all jobs that are currently waiting in the job queue.
*/
Collection<JobRun> getPendingJobs();
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/JobManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/JobManager.java
index 9850ee2..4882f4a 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/JobManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/job/JobManager.java
@@ -264,6 +264,11 @@
}
@Override
+ public int getRunningJobsCount() {
+ return activeRunMap.size();
+ }
+
+ @Override
public Collection<JobRun> getPendingJobs() {
return jobQueue.jobs();
}