|
32 | 32 | import org.apache.hadoop.fs.FileSystem; |
33 | 33 | import org.apache.hadoop.fs.Path; |
34 | 34 |
|
| 35 | +import io.opentelemetry.api.common.Attributes; |
| 36 | +import io.opentelemetry.api.metrics.Meter; |
35 | 37 | import lombok.Data; |
36 | 38 | import lombok.extern.slf4j.Slf4j; |
37 | 39 |
|
|
53 | 55 | import org.apache.gobblin.converter.initializer.ConverterInitializerFactory; |
54 | 56 | import org.apache.gobblin.initializer.Initializer; |
55 | 57 | import org.apache.gobblin.destination.DestinationDatasetHandlerService; |
| 58 | +import org.apache.gobblin.metrics.OpenTelemetryMetrics; |
| 59 | +import org.apache.gobblin.metrics.OpenTelemetryMetricsBase; |
| 60 | +import org.apache.gobblin.metrics.ServiceMetricNames; |
56 | 61 | import org.apache.gobblin.metrics.event.EventSubmitter; |
57 | 62 | import org.apache.gobblin.metrics.event.TimingEvent; |
58 | 63 | import org.apache.gobblin.runtime.AbstractJobLauncher; |
|
80 | 85 | import org.apache.gobblin.writer.initializer.WriterInitializer; |
81 | 86 | import org.apache.gobblin.writer.initializer.WriterInitializerFactory; |
82 | 87 |
|
| 88 | +import static org.apache.gobblin.runtime.JobState.GAAS_OBSERVABILITY_METRICS_GROUPNAME; |
| 89 | + |
83 | 90 |
|
84 | 91 |
|
85 | 92 | @Slf4j |
@@ -142,6 +149,8 @@ public GenerateWorkUnitsResult generateWorkUnits(Properties jobProps, EventSubmi |
142 | 149 | // TODO: provide for job cancellation (unless handling at the temporal-level of parent workflows)! |
143 | 150 | JobState jobState = new JobState(jobProps); |
144 | 151 | log.info("Created jobState: {}", jobState.toJsonString(true)); |
| 152 | + // emit jobs observed at RM level |
| 153 | + emitMetrics(jobState); |
145 | 154 |
|
146 | 155 | int heartBeatInterval = JobStateUtils.getHeartBeatInterval(jobState); |
147 | 156 | heartBeatExecutor.scheduleAtFixedRate(() -> activityExecutionContext.heartbeat("Running GenerateWorkUnits"), |
@@ -354,4 +363,37 @@ protected static EventTimer createWorkPreparedSizeDistillationTimer( |
354 | 363 | public static int getConfiguredNumSizeSummaryQuantiles(State state) { |
355 | 364 | return state.getPropAsInt(GenerateWorkUnits.NUM_WORK_UNITS_SIZE_SUMMARY_QUANTILES, GenerateWorkUnits.DEFAULT_NUM_WORK_UNITS_SIZE_SUMMARY_QUANTILES); |
356 | 365 | } |
| 366 | + |
| 367 | + /** |
| 368 | + * Emit metrics to indicate jobs observed at RM level |
| 369 | + * @param jobState job state |
| 370 | + */ |
| 371 | + private void emitMetrics(JobState jobState) { |
| 372 | + try { |
| 373 | + OpenTelemetryMetricsBase otelMetrics = OpenTelemetryMetrics.getInstance(jobState); |
| 374 | + if (otelMetrics == null) { |
| 375 | + log.warn("OpenTelemetry metrics instance is null, skipping metrics emission"); |
| 376 | + return; |
| 377 | + } |
| 378 | + |
| 379 | + Meter meter = otelMetrics.getMeter(GAAS_OBSERVABILITY_METRICS_GROUPNAME); |
| 380 | + Attributes tags = getEventAttributes(jobState); |
| 381 | + log.info("Emitting metrics for job: {}", jobState.getJobName()); |
| 382 | + String jobMetricDescription = "Number of Jobs observed on RM"; |
| 383 | + String jobMetricName = ServiceMetricNames.RM_JOB_OBSERVED_COUNT; |
| 384 | + meter.counterBuilder(jobMetricName).setDescription(jobMetricDescription).build().add(1, tags); |
| 385 | + } catch (Exception e) { |
| 386 | + log.error("Error in emitMetrics for job: {}", jobState.getJobName(), e); |
| 387 | + } |
| 388 | + } |
| 389 | + |
| 390 | + private Attributes getEventAttributes(JobState jobState) { |
| 391 | + return Attributes.builder() |
| 392 | + .put(TimingEvent.FlowEventConstants.FLOW_NAME_FIELD, jobState.getProp(ConfigurationKeys.FLOW_NAME_KEY, "NA")) |
| 393 | + .put(TimingEvent.FlowEventConstants.FLOW_GROUP_FIELD, jobState.getProp(ConfigurationKeys.FLOW_GROUP_KEY, "NA")) |
| 394 | + .put(TimingEvent.FlowEventConstants.FLOW_EXECUTION_ID_FIELD, jobState.getProp(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, "NA")) |
| 395 | + .put(TimingEvent.FlowEventConstants.FLOW_FABRIC, jobState.getProp(ConfigurationKeys.METRICS_REPORTING_OPENTELEMETRY_FABRIC, "NA")) |
| 396 | + .put(TimingEvent.FlowEventConstants.RM_HOST_FIELD, jobState.getProp(ConfigurationKeys.RM_HOST_KEY, "NA")) |
| 397 | + .build(); |
| 398 | + } |
357 | 399 | } |
0 commit comments