[ https://issues.apache.org/jira/browse/NIFI-4809?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16397712#comment-16397712 ]
ASF GitHub Bot commented on NIFI-4809: -------------------------------------- Github user mattyb149 commented on a diff in the pull request: https://github.com/apache/nifi/pull/2430#discussion_r174294045 --- Diff: nifi-nar-bundles/nifi-site-to-site-reporting-bundle/nifi-site-to-site-reporting-task/src/main/java/org/apache/nifi/reporting/SiteToSiteMetricsReportingTask.java --- @@ -0,0 +1,212 @@ +/* + * 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.nifi.reporting; + +import java.io.ByteArrayInputStream; +import java.io.IOException; +import java.io.InputStream; +import java.lang.management.ManagementFactory; +import java.lang.management.OperatingSystemMXBean; +import java.nio.charset.StandardCharsets; +import java.util.ArrayList; +import java.util.Collection; +import java.util.Collections; +import java.util.HashMap; +import java.util.List; +import java.util.Map; +import java.util.UUID; +import java.util.concurrent.TimeUnit; + +import javax.json.Json; +import javax.json.JsonBuilderFactory; +import javax.json.JsonObject; + +import org.apache.avro.Schema; +import org.apache.nifi.annotation.documentation.CapabilityDescription; +import org.apache.nifi.annotation.documentation.Tags; +import org.apache.nifi.avro.AvroTypeUtil; +import org.apache.nifi.components.AllowableValue; +import org.apache.nifi.components.PropertyDescriptor; +import org.apache.nifi.components.ValidationContext; +import org.apache.nifi.components.ValidationResult; +import org.apache.nifi.components.Validator; +import org.apache.nifi.controller.status.ProcessGroupStatus; +import org.apache.nifi.flowfile.attributes.CoreAttributes; +import org.apache.nifi.processor.exception.ProcessException; +import org.apache.nifi.processor.util.StandardValidators; +import org.apache.nifi.remote.Transaction; +import org.apache.nifi.remote.TransferDirection; +import org.apache.nifi.reporting.util.metrics.MetricNames; +import org.apache.nifi.reporting.util.metrics.MetricsService; +import org.apache.nifi.reporting.util.metrics.api.MetricsBuilder; + +import com.yammer.metrics.core.VirtualMachineMetrics; + +@Tags({"status", "metrics", "site", "site to site"}) +@CapabilityDescription("Publishes same metrics as the Ambari Reporting task using the Site To Site protocol.") +public class SiteToSiteMetricsReportingTask extends AbstractSiteToSiteReportingTask { + + static final AllowableValue AMBARI_FORMAT = new AllowableValue("ambari-format", "Ambari Format", "Metrics will be formatted" + + " according to the Ambari Metrics API. See Additional Details in Usage documentation."); + static final AllowableValue RECORD_FORMAT = new AllowableValue("record-format", "Record Format", "Metrics will be formatted" + + " using the Record Writer property of this reporting task. See Additional Details in Usage documentation to" + + " have the description of the default schema."); + + static final PropertyDescriptor APPLICATION_ID = new PropertyDescriptor.Builder() + .name("s2s-metrics-application-id") + .displayName("Application ID") + .description("The Application ID to be included in the metrics") + .required(true) + .expressionLanguageSupported(true) + .defaultValue("nifi") + .addValidator(StandardValidators.NON_EMPTY_VALIDATOR) + .build(); + + static final PropertyDescriptor HOSTNAME = new PropertyDescriptor.Builder() + .name("s2s-metrics-hostname") + .displayName("Hostname") + .description("The Hostname of this NiFi instance to be included in the metrics") + .required(true) + .expressionLanguageSupported(true) + .defaultValue("${hostname(true)}") + .addValidator(StandardValidators.NON_EMPTY_VALIDATOR) + .build(); + + static final PropertyDescriptor FORMAT = new PropertyDescriptor.Builder() + .name("s2s-metrics-format") + .displayName("Output format") + .description("The output format that will be used for the metrics") + .required(true) + .allowableValues(AMBARI_FORMAT, RECORD_FORMAT) + .defaultValue(AMBARI_FORMAT.getValue()) + .addValidator(Validator.VALID) + .build(); + + private final MetricsService metricsService = new MetricsService(); + + public SiteToSiteMetricsReportingTask() throws IOException { + final InputStream schema = getClass().getClassLoader().getResourceAsStream("schema-metrics.avsc"); + recordSchema = AvroTypeUtil.createSchema(new Schema.Parser().parse(schema)); + } + + @Override + protected List<PropertyDescriptor> getSupportedPropertyDescriptors() { + final List<PropertyDescriptor> properties = new ArrayList<>(super.getSupportedPropertyDescriptors()); + properties.add(HOSTNAME); + properties.add(APPLICATION_ID); + properties.add(FORMAT); + properties.add(RECORD_WRITER); + properties.remove(BATCH_SIZE); + return properties; + } + + @Override + protected Collection<ValidationResult> customValidate(ValidationContext validationContext) { + final List<ValidationResult> problems = new ArrayList<>(super.customValidate(validationContext)); + + final boolean isWriterSet = validationContext.getProperty(RECORD_WRITER).isSet(); + if (validationContext.getProperty(FORMAT).getValue().equals(RECORD_FORMAT.getValue()) && !isWriterSet) { --- End diff -- Can we also check that a Record Writer is NOT set when Ambari Format is selected? This isn't a requirement, just a suggestion. Since it requires two operations to switch back and forth between formats, that's not an ideal user experience, so I'm fine with whichever way you'd like to do it :) > Implement a SiteToSiteMetricsReportingTask > ------------------------------------------ > > Key: NIFI-4809 > URL: https://issues.apache.org/jira/browse/NIFI-4809 > Project: Apache NiFi > Issue Type: New Feature > Components: Extensions > Reporter: Pierre Villard > Assignee: Pierre Villard > Priority: Major > > At the moment there is an AmbariReportingTask to send the NiFi-related > metrics of the host to the Ambari Metrics Service. In a multi-cluster > configuration, or when working with MiNiFi (Java) agents, it might not be > possible for all the NiFi instances (NiFi and/or MiNiFi) to access the AMS > REST API. > To solve this problem, a solution would be to implement a > SiteToSiteMetricsReportingTask to send the data via S2S to the "main" NiFi > instance/cluster that will be able to publish the metrics into AMS (using > InvokeHTTP). This way, it is possible to have the metrics of all the > instances exposed in one AMS instance. > I propose to send the data formatted as we are doing right now in the Ambari > reporting task. If needed, it can be easily converted into another schema > using the record processors once received via S2S. -- This message was sent by Atlassian JIRA (v7.6.3#76005)