mirror of https://github.com/apache/nifi.git
NIFI-4280:
- Adding support for the user to configure variables in the UI. - Updating the endpoints for changing variables as necessary. This closes #2135.
This commit is contained in:
parent
91383264d8
commit
eac47e90cb
|
@ -17,43 +17,101 @@
|
||||||
|
|
||||||
package org.apache.nifi.web.api.dto;
|
package org.apache.nifi.web.api.dto;
|
||||||
|
|
||||||
import javax.xml.bind.annotation.XmlType;
|
|
||||||
|
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
|
|
||||||
|
import javax.xml.bind.annotation.XmlType;
|
||||||
|
import java.util.Collection;
|
||||||
|
|
||||||
@XmlType(name = "affectedComponent")
|
@XmlType(name = "affectedComponent")
|
||||||
public class AffectedComponentDTO {
|
public class AffectedComponentDTO {
|
||||||
public static final String COMPONENT_TYPE_PROCESSOR = "PROCESSOR";
|
public static final String COMPONENT_TYPE_PROCESSOR = "PROCESSOR";
|
||||||
public static final String COMPONENT_TYPE_CONTROLLER_SERVICE = "CONTROLLER_SERVICE";
|
public static final String COMPONENT_TYPE_CONTROLLER_SERVICE = "CONTROLLER_SERVICE";
|
||||||
|
|
||||||
private String parentGroupId;
|
private String processGroupId;
|
||||||
private String componentId;
|
private String id;
|
||||||
private String componentType;
|
private String referenceType;
|
||||||
|
private String name;
|
||||||
|
private String state;
|
||||||
|
private Integer activeThreadCount;
|
||||||
|
|
||||||
|
private Collection<String> validationErrors;
|
||||||
|
|
||||||
@ApiModelProperty("The UUID of the Process Group that this component is in")
|
@ApiModelProperty("The UUID of the Process Group that this component is in")
|
||||||
public String getParentGroupId() {
|
public String getProcessGroupId() {
|
||||||
return parentGroupId;
|
return processGroupId;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setParentGroupId(final String parentGroupId) {
|
public void setProcessGroupId(final String processGroupId) {
|
||||||
this.parentGroupId = parentGroupId;
|
this.processGroupId = processGroupId;
|
||||||
}
|
}
|
||||||
|
|
||||||
@ApiModelProperty("The UUID of this component")
|
@ApiModelProperty("The UUID of this component")
|
||||||
public String getComponentId() {
|
public String getId() {
|
||||||
return componentId;
|
return id;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setComponentId(final String componentId) {
|
public void setId(final String id) {
|
||||||
this.componentId = componentId;
|
this.id = id;
|
||||||
}
|
}
|
||||||
|
|
||||||
@ApiModelProperty(value = "The type of this component", allowableValues = COMPONENT_TYPE_PROCESSOR + "," + COMPONENT_TYPE_CONTROLLER_SERVICE)
|
@ApiModelProperty(value = "The type of this component", allowableValues = COMPONENT_TYPE_PROCESSOR + "," + COMPONENT_TYPE_CONTROLLER_SERVICE)
|
||||||
public String getComponentType() {
|
public String getReferenceType() {
|
||||||
return componentType;
|
return referenceType;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setComponentType(final String componentType) {
|
public void setReferenceType(final String referenceType) {
|
||||||
this.componentType = componentType;
|
this.referenceType = referenceType;
|
||||||
|
}
|
||||||
|
|
||||||
|
@ApiModelProperty("The name of this component.")
|
||||||
|
public String getName() {
|
||||||
|
return name;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setName(String name) {
|
||||||
|
this.name = name;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @return scheduled state of the processor referencing a controller service. If this component is another service, this field represents the controller service state
|
||||||
|
*/
|
||||||
|
@ApiModelProperty(
|
||||||
|
value = "The scheduled state of a processor or reporting task referencing a controller service. If this component is another controller "
|
||||||
|
+ "service, this field represents the controller service state."
|
||||||
|
)
|
||||||
|
public String getState() {
|
||||||
|
return state;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setState(String state) {
|
||||||
|
this.state = state;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @return active thread count for the referencing component
|
||||||
|
*/
|
||||||
|
@ApiModelProperty(
|
||||||
|
value = "The number of active threads for the referencing component."
|
||||||
|
)
|
||||||
|
public Integer getActiveThreadCount() {
|
||||||
|
return activeThreadCount;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setActiveThreadCount(Integer activeThreadCount) {
|
||||||
|
this.activeThreadCount = activeThreadCount;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @return Any validation error associated with this component
|
||||||
|
*/
|
||||||
|
@ApiModelProperty(
|
||||||
|
value = "The validation errors for the component."
|
||||||
|
)
|
||||||
|
public Collection<String> getValidationErrors() {
|
||||||
|
return validationErrors;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setValidationErrors(Collection<String> validationErrors) {
|
||||||
|
this.validationErrors = validationErrors;
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -19,10 +19,10 @@ package org.apache.nifi.web.api.dto;
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
import org.apache.nifi.web.api.entity.ControllerServiceReferencingComponentEntity;
|
import org.apache.nifi.web.api.entity.ControllerServiceReferencingComponentEntity;
|
||||||
|
|
||||||
|
import javax.xml.bind.annotation.XmlType;
|
||||||
import java.util.Collection;
|
import java.util.Collection;
|
||||||
import java.util.Map;
|
import java.util.Map;
|
||||||
import java.util.Set;
|
import java.util.Set;
|
||||||
import javax.xml.bind.annotation.XmlType;
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* A component referencing a controller service. This can either be another controller service or a processor. Depending on the type of component different properties may be set.
|
* A component referencing a controller service. This can either be another controller service or a processor. Depending on the type of component different properties may be set.
|
||||||
|
@ -105,11 +105,11 @@ public class ControllerServiceReferencingComponentDTO {
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* @return state of the processor referencing a controller service. If this component is another service, this field is blank
|
* @return scheduled state of the processor referencing a controller service. If this component is another service, this field represents the controller service state
|
||||||
*/
|
*/
|
||||||
@ApiModelProperty(
|
@ApiModelProperty(
|
||||||
value = "The state of a processor or reporting task referencing a controller service. If this component is another controller "
|
value = "The scheduled state of a processor or reporting task referencing a controller service. If this component is another controller "
|
||||||
+ "service, this field is blank."
|
+ "service, this field represents the controller service state."
|
||||||
)
|
)
|
||||||
public String getState() {
|
public String getState() {
|
||||||
return state;
|
return state;
|
||||||
|
|
|
@ -17,19 +17,18 @@
|
||||||
|
|
||||||
package org.apache.nifi.web.api.dto;
|
package org.apache.nifi.web.api.dto;
|
||||||
|
|
||||||
import java.util.HashSet;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
import java.util.Set;
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
|
|
||||||
import javax.xml.bind.annotation.XmlType;
|
import javax.xml.bind.annotation.XmlType;
|
||||||
|
import java.util.Set;
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
|
||||||
|
|
||||||
@XmlType(name = "variable")
|
@XmlType(name = "variable")
|
||||||
public class VariableDTO {
|
public class VariableDTO {
|
||||||
private String name;
|
private String name;
|
||||||
private String value;
|
private String value;
|
||||||
private String processGroupId;
|
private String processGroupId;
|
||||||
private Set<AffectedComponentDTO> affectedComponents = new HashSet<>();
|
private Set<AffectedComponentEntity> affectedComponents;
|
||||||
|
|
||||||
@ApiModelProperty("The name of the variable")
|
@ApiModelProperty("The name of the variable")
|
||||||
public String getName() {
|
public String getName() {
|
||||||
|
@ -59,11 +58,11 @@ public class VariableDTO {
|
||||||
}
|
}
|
||||||
|
|
||||||
@ApiModelProperty(value = "A set of all components that will be affected if the value of this variable is changed", readOnly = true)
|
@ApiModelProperty(value = "A set of all components that will be affected if the value of this variable is changed", readOnly = true)
|
||||||
public Set<AffectedComponentDTO> getAffectedComponents() {
|
public Set<AffectedComponentEntity> getAffectedComponents() {
|
||||||
return affectedComponents;
|
return affectedComponents;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setAffectedComponents(Set<AffectedComponentDTO> affectedComponents) {
|
public void setAffectedComponents(Set<AffectedComponentEntity> affectedComponents) {
|
||||||
this.affectedComponents = affectedComponents;
|
this.affectedComponents = affectedComponents;
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -17,18 +17,16 @@
|
||||||
|
|
||||||
package org.apache.nifi.web.api.dto;
|
package org.apache.nifi.web.api.dto;
|
||||||
|
|
||||||
import java.util.Set;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
|
|
||||||
import javax.xml.bind.annotation.XmlType;
|
|
||||||
|
|
||||||
import org.apache.nifi.web.api.entity.VariableEntity;
|
import org.apache.nifi.web.api.entity.VariableEntity;
|
||||||
|
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
import javax.xml.bind.annotation.XmlType;
|
||||||
|
import java.util.Set;
|
||||||
|
|
||||||
@XmlType(name = "variableRegistry")
|
@XmlType(name = "variableRegistry")
|
||||||
public class VariableRegistryDTO {
|
public class VariableRegistryDTO {
|
||||||
private Set<VariableEntity> variables;
|
private Set<VariableEntity> variables;
|
||||||
private String groupId;
|
private String processGroupId;
|
||||||
|
|
||||||
public void setVariables(final Set<VariableEntity> variables) {
|
public void setVariables(final Set<VariableEntity> variables) {
|
||||||
this.variables = variables;
|
this.variables = variables;
|
||||||
|
@ -39,12 +37,12 @@ public class VariableRegistryDTO {
|
||||||
return variables;
|
return variables;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setProcessGroupId(final String groupId) {
|
public void setProcessGroupId(final String processGroupId) {
|
||||||
this.groupId = groupId;
|
this.processGroupId = processGroupId;
|
||||||
}
|
}
|
||||||
|
|
||||||
@ApiModelProperty("The UUID of the Process Group that this Variable Registry belongs to")
|
@ApiModelProperty("The UUID of the Process Group that this Variable Registry belongs to")
|
||||||
public String getProcessGroupId() {
|
public String getProcessGroupId() {
|
||||||
return groupId;
|
return processGroupId;
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -17,27 +17,27 @@
|
||||||
|
|
||||||
package org.apache.nifi.web.api.dto;
|
package org.apache.nifi.web.api.dto;
|
||||||
|
|
||||||
import java.util.Date;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
import java.util.List;
|
import org.apache.nifi.web.api.dto.util.TimestampAdapter;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
|
|
||||||
import javax.xml.bind.annotation.XmlType;
|
import javax.xml.bind.annotation.XmlType;
|
||||||
import javax.xml.bind.annotation.adapters.XmlJavaTypeAdapter;
|
import javax.xml.bind.annotation.adapters.XmlJavaTypeAdapter;
|
||||||
|
import java.util.Date;
|
||||||
import org.apache.nifi.web.api.dto.util.TimestampAdapter;
|
import java.util.List;
|
||||||
|
import java.util.Set;
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
|
||||||
|
|
||||||
@XmlType(name = "variableRegistryUpdateRequest")
|
@XmlType(name = "variableRegistryUpdateRequest")
|
||||||
public class VariableRegistryUpdateRequestDTO {
|
public class VariableRegistryUpdateRequestDTO {
|
||||||
private String requestId;
|
private String requestId;
|
||||||
private String processGroupId;
|
private String processGroupId;
|
||||||
private String uri;
|
private String uri;
|
||||||
private Date submissionTime = new Date();
|
private Date submissionTime;
|
||||||
private Date lastUpdated = new Date();
|
private Date lastUpdated;
|
||||||
private boolean complete = false;
|
private boolean complete = false;
|
||||||
private String failureReason;
|
private String failureReason;
|
||||||
private List<VariableRegistryUpdateStepDTO> updateSteps;
|
private List<VariableRegistryUpdateStepDTO> updateSteps;
|
||||||
|
private Set<AffectedComponentEntity> affectedComponents;
|
||||||
|
|
||||||
@ApiModelProperty("The unique ID of the Process Group that the variable registry belongs to")
|
@ApiModelProperty("The unique ID of the Process Group that the variable registry belongs to")
|
||||||
public String getProcessGroupId() {
|
public String getProcessGroupId() {
|
||||||
|
@ -112,4 +112,13 @@ public class VariableRegistryUpdateRequestDTO {
|
||||||
public void setFailureReason(String reason) {
|
public void setFailureReason(String reason) {
|
||||||
this.failureReason = reason;
|
this.failureReason = reason;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ApiModelProperty(value = "A set of all components that will be affected if the value of this variable is changed", readOnly = true)
|
||||||
|
public Set<AffectedComponentEntity> getAffectedComponents() {
|
||||||
|
return affectedComponents;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setAffectedComponents(Set<AffectedComponentEntity> affectedComponents) {
|
||||||
|
this.affectedComponents = affectedComponents;
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -0,0 +1,44 @@
|
||||||
|
/*
|
||||||
|
* 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.web.api.entity;
|
||||||
|
|
||||||
|
import org.apache.nifi.web.api.dto.AffectedComponentDTO;
|
||||||
|
|
||||||
|
import javax.xml.bind.annotation.XmlRootElement;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* A serialized representation of this class can be placed in the entity body of a response to the API.
|
||||||
|
* This particular entity holds a reference to component that references a variable.
|
||||||
|
*/
|
||||||
|
@XmlRootElement(name = "affectComponentEntity")
|
||||||
|
public class AffectedComponentEntity extends ComponentEntity implements Permissible<AffectedComponentDTO> {
|
||||||
|
|
||||||
|
private AffectedComponentDTO component;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @return variable referencing components that is being serialized
|
||||||
|
*/
|
||||||
|
public AffectedComponentDTO getComponent() {
|
||||||
|
return component;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setComponent(AffectedComponentDTO component) {
|
||||||
|
this.component = component;
|
||||||
|
}
|
||||||
|
|
||||||
|
}
|
|
@ -17,16 +17,15 @@
|
||||||
|
|
||||||
package org.apache.nifi.web.api.entity;
|
package org.apache.nifi.web.api.entity;
|
||||||
|
|
||||||
import javax.xml.bind.annotation.XmlRootElement;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
|
|
||||||
import org.apache.nifi.web.api.dto.RevisionDTO;
|
import org.apache.nifi.web.api.dto.RevisionDTO;
|
||||||
import org.apache.nifi.web.api.dto.VariableRegistryDTO;
|
import org.apache.nifi.web.api.dto.VariableRegistryDTO;
|
||||||
|
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
import javax.xml.bind.annotation.XmlRootElement;
|
||||||
|
|
||||||
@XmlRootElement(name = "variableRegistryEntity")
|
@XmlRootElement(name = "variableRegistryEntity")
|
||||||
public class VariableRegistryEntity extends Entity {
|
public class VariableRegistryEntity extends Entity {
|
||||||
private RevisionDTO groupRevision;
|
private RevisionDTO processGroupRevision;
|
||||||
private VariableRegistryDTO variableRegistry;
|
private VariableRegistryDTO variableRegistry;
|
||||||
|
|
||||||
|
|
||||||
|
@ -41,10 +40,10 @@ public class VariableRegistryEntity extends Entity {
|
||||||
|
|
||||||
@ApiModelProperty("The revision of the Process Group that the Variable Registry belongs to")
|
@ApiModelProperty("The revision of the Process Group that the Variable Registry belongs to")
|
||||||
public RevisionDTO getProcessGroupRevision() {
|
public RevisionDTO getProcessGroupRevision() {
|
||||||
return groupRevision;
|
return processGroupRevision;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setProcessGroupRevision(RevisionDTO revision) {
|
public void setProcessGroupRevision(RevisionDTO processGroupRevision) {
|
||||||
this.groupRevision = revision;
|
this.processGroupRevision = processGroupRevision;
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -17,16 +17,15 @@
|
||||||
|
|
||||||
package org.apache.nifi.web.api.entity;
|
package org.apache.nifi.web.api.entity;
|
||||||
|
|
||||||
import javax.xml.bind.annotation.XmlRootElement;
|
import com.wordnik.swagger.annotations.ApiModelProperty;
|
||||||
|
|
||||||
import org.apache.nifi.web.api.dto.RevisionDTO;
|
import org.apache.nifi.web.api.dto.RevisionDTO;
|
||||||
import org.apache.nifi.web.api.dto.VariableRegistryUpdateRequestDTO;
|
import org.apache.nifi.web.api.dto.VariableRegistryUpdateRequestDTO;
|
||||||
|
|
||||||
import com.wordnik.swagger.annotations.ApiModelProperty;
|
import javax.xml.bind.annotation.XmlRootElement;
|
||||||
|
|
||||||
@XmlRootElement(name = "variableRegistryUpdateRequestEntity")
|
@XmlRootElement(name = "variableRegistryUpdateRequestEntity")
|
||||||
public class VariableRegistryUpdateRequestEntity extends Entity {
|
public class VariableRegistryUpdateRequestEntity extends Entity {
|
||||||
private VariableRegistryUpdateRequestDTO requestDto;
|
private VariableRegistryUpdateRequestDTO request;
|
||||||
private RevisionDTO processGroupRevision;
|
private RevisionDTO processGroupRevision;
|
||||||
|
|
||||||
@ApiModelProperty("The revision for the Process Group that owns this variable registry.")
|
@ApiModelProperty("The revision for the Process Group that owns this variable registry.")
|
||||||
|
@ -39,11 +38,11 @@ public class VariableRegistryUpdateRequestEntity extends Entity {
|
||||||
}
|
}
|
||||||
|
|
||||||
@ApiModelProperty("The Variable Registry Update Request")
|
@ApiModelProperty("The Variable Registry Update Request")
|
||||||
public VariableRegistryUpdateRequestDTO getRequestDto() {
|
public VariableRegistryUpdateRequestDTO getRequest() {
|
||||||
return requestDto;
|
return request;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void setRequestDto(VariableRegistryUpdateRequestDTO requestDto) {
|
public void setRequest(VariableRegistryUpdateRequestDTO request) {
|
||||||
this.requestDto = requestDto;
|
this.request = request;
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -16,16 +16,6 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.nifi.cluster.coordination.http;
|
package org.apache.nifi.cluster.coordination.http;
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
import java.net.URI;
|
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Set;
|
|
||||||
import java.util.concurrent.TimeUnit;
|
|
||||||
import java.util.stream.Collectors;
|
|
||||||
|
|
||||||
import javax.ws.rs.core.StreamingOutput;
|
|
||||||
|
|
||||||
import org.apache.nifi.cluster.coordination.http.endpoints.AccessPolicyEndpointMerger;
|
import org.apache.nifi.cluster.coordination.http.endpoints.AccessPolicyEndpointMerger;
|
||||||
import org.apache.nifi.cluster.coordination.http.endpoints.BulletinBoardEndpointMerger;
|
import org.apache.nifi.cluster.coordination.http.endpoints.BulletinBoardEndpointMerger;
|
||||||
import org.apache.nifi.cluster.coordination.http.endpoints.ComponentStateEndpointMerger;
|
import org.apache.nifi.cluster.coordination.http.endpoints.ComponentStateEndpointMerger;
|
||||||
|
@ -79,6 +69,7 @@ import org.apache.nifi.cluster.coordination.http.endpoints.UserEndpointMerger;
|
||||||
import org.apache.nifi.cluster.coordination.http.endpoints.UserGroupEndpointMerger;
|
import org.apache.nifi.cluster.coordination.http.endpoints.UserGroupEndpointMerger;
|
||||||
import org.apache.nifi.cluster.coordination.http.endpoints.UserGroupsEndpointMerger;
|
import org.apache.nifi.cluster.coordination.http.endpoints.UserGroupsEndpointMerger;
|
||||||
import org.apache.nifi.cluster.coordination.http.endpoints.UsersEndpointMerger;
|
import org.apache.nifi.cluster.coordination.http.endpoints.UsersEndpointMerger;
|
||||||
|
import org.apache.nifi.cluster.coordination.http.endpoints.VariableRegistryEndpointMerger;
|
||||||
import org.apache.nifi.cluster.coordination.http.replication.RequestReplicator;
|
import org.apache.nifi.cluster.coordination.http.replication.RequestReplicator;
|
||||||
import org.apache.nifi.cluster.manager.NodeResponse;
|
import org.apache.nifi.cluster.manager.NodeResponse;
|
||||||
import org.apache.nifi.stream.io.NullOutputStream;
|
import org.apache.nifi.stream.io.NullOutputStream;
|
||||||
|
@ -87,6 +78,15 @@ import org.apache.nifi.util.NiFiProperties;
|
||||||
import org.slf4j.Logger;
|
import org.slf4j.Logger;
|
||||||
import org.slf4j.LoggerFactory;
|
import org.slf4j.LoggerFactory;
|
||||||
|
|
||||||
|
import javax.ws.rs.core.StreamingOutput;
|
||||||
|
import java.io.IOException;
|
||||||
|
import java.net.URI;
|
||||||
|
import java.util.ArrayList;
|
||||||
|
import java.util.List;
|
||||||
|
import java.util.Set;
|
||||||
|
import java.util.concurrent.TimeUnit;
|
||||||
|
import java.util.stream.Collectors;
|
||||||
|
|
||||||
public class StandardHttpResponseMapper implements HttpResponseMapper {
|
public class StandardHttpResponseMapper implements HttpResponseMapper {
|
||||||
|
|
||||||
private Logger logger = LoggerFactory.getLogger(StandardHttpResponseMapper.class);
|
private Logger logger = LoggerFactory.getLogger(StandardHttpResponseMapper.class);
|
||||||
|
@ -154,6 +154,7 @@ public class StandardHttpResponseMapper implements HttpResponseMapper {
|
||||||
endpointMergers.add(new UserGroupEndpointMerger());
|
endpointMergers.add(new UserGroupEndpointMerger());
|
||||||
endpointMergers.add(new AccessPolicyEndpointMerger());
|
endpointMergers.add(new AccessPolicyEndpointMerger());
|
||||||
endpointMergers.add(new SearchUsersEndpointMerger());
|
endpointMergers.add(new SearchUsersEndpointMerger());
|
||||||
|
endpointMergers.add(new VariableRegistryEndpointMerger());
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
|
@ -0,0 +1,113 @@
|
||||||
|
/*
|
||||||
|
* 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.cluster.coordination.http.endpoints;
|
||||||
|
|
||||||
|
import org.apache.nifi.cluster.coordination.http.EndpointResponseMerger;
|
||||||
|
import org.apache.nifi.cluster.manager.AffectedComponentEntityMerger;
|
||||||
|
import org.apache.nifi.cluster.manager.NodeResponse;
|
||||||
|
import org.apache.nifi.cluster.protocol.NodeIdentifier;
|
||||||
|
import org.apache.nifi.web.api.dto.VariableDTO;
|
||||||
|
import org.apache.nifi.web.api.dto.VariableRegistryDTO;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
|
import org.apache.nifi.web.api.entity.VariableEntity;
|
||||||
|
import org.apache.nifi.web.api.entity.VariableRegistryEntity;
|
||||||
|
|
||||||
|
import java.net.URI;
|
||||||
|
import java.util.HashMap;
|
||||||
|
import java.util.Iterator;
|
||||||
|
import java.util.Map;
|
||||||
|
import java.util.Set;
|
||||||
|
import java.util.regex.Pattern;
|
||||||
|
|
||||||
|
public class VariableRegistryEndpointMerger extends AbstractSingleEntityEndpoint<VariableRegistryEntity> implements EndpointResponseMerger {
|
||||||
|
public static final Pattern VARIABLE_REGISTRY_UPDATE_REQUEST_URI_PATTERN = Pattern.compile("/nifi-api/process-groups/(?:(?:root)|(?:[a-f0-9\\-]{36}))/variable-registry");
|
||||||
|
|
||||||
|
private final AffectedComponentEntityMerger affectedComponentEntityMerger = new AffectedComponentEntityMerger();
|
||||||
|
|
||||||
|
@Override
|
||||||
|
public boolean canHandle(final URI uri, final String method) {
|
||||||
|
if (("GET".equalsIgnoreCase(method) || "PUT".equalsIgnoreCase(method)) && (VARIABLE_REGISTRY_UPDATE_REQUEST_URI_PATTERN.matcher(uri.getPath()).matches())) {
|
||||||
|
return true;
|
||||||
|
}
|
||||||
|
|
||||||
|
return false;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
protected Class<VariableRegistryEntity> getEntityClass() {
|
||||||
|
return VariableRegistryEntity.class;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
protected void mergeResponses(final VariableRegistryEntity clientEntity, final Map<NodeIdentifier, VariableRegistryEntity> entityMap,
|
||||||
|
final Set<NodeResponse> successfulResponses, final Set<NodeResponse> problematicResponses) {
|
||||||
|
|
||||||
|
final VariableRegistryDTO clientVariableRegistry = clientEntity.getVariableRegistry();
|
||||||
|
final Set<VariableEntity> clientVariableEntities = clientVariableRegistry.getVariables();
|
||||||
|
|
||||||
|
if (clientVariableEntities != null) {
|
||||||
|
for (final Iterator<VariableEntity> i = clientVariableEntities.iterator(); i.hasNext();) {
|
||||||
|
final VariableEntity clientVariableEntity = i.next();
|
||||||
|
final VariableDTO clientVariable = clientVariableEntity.getVariable();
|
||||||
|
|
||||||
|
final Map<NodeIdentifier, Set<AffectedComponentEntity>> nodeAffectedComponentEntities = new HashMap<>();
|
||||||
|
|
||||||
|
boolean retainClientVariable = true;
|
||||||
|
for (final Map.Entry<NodeIdentifier, VariableRegistryEntity> nodeEntry : entityMap.entrySet()) {
|
||||||
|
final VariableRegistryEntity nodeVariableRegistry = nodeEntry.getValue();
|
||||||
|
final Set<VariableEntity> nodeVariableEntities = nodeVariableRegistry.getVariableRegistry().getVariables();
|
||||||
|
|
||||||
|
// if this node has no variables, then the current client variable should be removed
|
||||||
|
if (nodeVariableEntities == null) {
|
||||||
|
retainClientVariable = false;
|
||||||
|
break;
|
||||||
|
}
|
||||||
|
|
||||||
|
boolean variableFound = false;
|
||||||
|
for (final VariableEntity nodeVariableEntity : nodeVariableEntities) {
|
||||||
|
final VariableDTO nodeVariable = nodeVariableEntity.getVariable();
|
||||||
|
|
||||||
|
// identify the current clientVariable for each node
|
||||||
|
if (clientVariable.getProcessGroupId().equals(nodeVariable.getProcessGroupId()) && clientVariable.getName().equals(nodeVariable.getName())) {
|
||||||
|
variableFound = true;
|
||||||
|
|
||||||
|
if (Boolean.FALSE.equals(nodeVariableEntity.getCanWrite())) {
|
||||||
|
clientVariableEntity.setCanWrite(false);
|
||||||
|
}
|
||||||
|
|
||||||
|
nodeAffectedComponentEntities.put(nodeEntry.getKey(), nodeVariableEntity.getVariable().getAffectedComponents());
|
||||||
|
break;
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
if (!variableFound) {
|
||||||
|
retainClientVariable = false;
|
||||||
|
break;
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
if (!retainClientVariable) {
|
||||||
|
i.remove();
|
||||||
|
} else {
|
||||||
|
final Set<AffectedComponentEntity> clientAffectedComponentEntities = clientVariableEntity.getVariable().getAffectedComponents();
|
||||||
|
affectedComponentEntityMerger.mergeAffectedComponents(clientAffectedComponentEntities, nodeAffectedComponentEntities);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,102 @@
|
||||||
|
/*
|
||||||
|
* 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.cluster.manager;
|
||||||
|
|
||||||
|
import org.apache.nifi.cluster.protocol.NodeIdentifier;
|
||||||
|
import org.apache.nifi.controller.service.ControllerServiceState;
|
||||||
|
import org.apache.nifi.web.api.dto.AffectedComponentDTO;
|
||||||
|
import org.apache.nifi.web.api.dto.PermissionsDTO;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
|
|
||||||
|
import java.util.HashMap;
|
||||||
|
import java.util.Map;
|
||||||
|
import java.util.Set;
|
||||||
|
|
||||||
|
public class AffectedComponentEntityMerger {
|
||||||
|
|
||||||
|
public void mergeAffectedComponents(final Set<AffectedComponentEntity> affectedComponents, final Map<NodeIdentifier, Set<AffectedComponentEntity>> affectedComponentMap) {
|
||||||
|
|
||||||
|
final Map<String, Integer> activeThreadCounts = new HashMap<>();
|
||||||
|
final Map<String, String> states = new HashMap<>();
|
||||||
|
final Map<String, PermissionsDTO> canReads = new HashMap<>();
|
||||||
|
|
||||||
|
for (final Map.Entry<NodeIdentifier, Set<AffectedComponentEntity>> nodeEntry : affectedComponentMap.entrySet()) {
|
||||||
|
final Set<AffectedComponentEntity> nodeAffectedComponents = nodeEntry.getValue();
|
||||||
|
|
||||||
|
// go through all the nodes referencing components
|
||||||
|
if (nodeAffectedComponents != null) {
|
||||||
|
for (final AffectedComponentEntity nodeAffectedComponentEntity : nodeAffectedComponents) {
|
||||||
|
final AffectedComponentDTO nodeAffectedComponent = nodeAffectedComponentEntity.getComponent();
|
||||||
|
|
||||||
|
if (nodeAffectedComponentEntity.getPermissions().getCanRead()) {
|
||||||
|
// handle active thread counts
|
||||||
|
if (nodeAffectedComponent.getActiveThreadCount() != null && nodeAffectedComponent.getActiveThreadCount() > 0) {
|
||||||
|
final Integer current = activeThreadCounts.get(nodeAffectedComponent.getId());
|
||||||
|
if (current == null) {
|
||||||
|
activeThreadCounts.put(nodeAffectedComponent.getId(), nodeAffectedComponent.getActiveThreadCount());
|
||||||
|
} else {
|
||||||
|
activeThreadCounts.put(nodeAffectedComponent.getId(), nodeAffectedComponent.getActiveThreadCount() + current);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// handle controller service state
|
||||||
|
final String state = states.get(nodeAffectedComponent.getId());
|
||||||
|
if (state == null) {
|
||||||
|
if (ControllerServiceState.DISABLING.name().equals(nodeAffectedComponent.getState())) {
|
||||||
|
states.put(nodeAffectedComponent.getId(), ControllerServiceState.DISABLING.name());
|
||||||
|
} else if (ControllerServiceState.ENABLING.name().equals(nodeAffectedComponent.getState())) {
|
||||||
|
states.put(nodeAffectedComponent.getId(), ControllerServiceState.ENABLING.name());
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// handle read permissions
|
||||||
|
final PermissionsDTO mergedPermissions = canReads.get(nodeAffectedComponentEntity.getId());
|
||||||
|
final PermissionsDTO permissions = nodeAffectedComponentEntity.getPermissions();
|
||||||
|
if (permissions != null) {
|
||||||
|
if (mergedPermissions == null) {
|
||||||
|
canReads.put(nodeAffectedComponentEntity.getId(), permissions);
|
||||||
|
} else {
|
||||||
|
PermissionsDtoMerger.mergePermissions(mergedPermissions, permissions);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// go through each affected components
|
||||||
|
if (affectedComponents != null) {
|
||||||
|
for (final AffectedComponentEntity affectedComponent : affectedComponents) {
|
||||||
|
final PermissionsDTO permissions = canReads.get(affectedComponent.getId());
|
||||||
|
if (permissions != null && permissions.getCanRead() != null && permissions.getCanRead()) {
|
||||||
|
final Integer activeThreadCount = activeThreadCounts.get(affectedComponent.getId());
|
||||||
|
if (activeThreadCount != null) {
|
||||||
|
affectedComponent.getComponent().setActiveThreadCount(activeThreadCount);
|
||||||
|
}
|
||||||
|
|
||||||
|
final String state = states.get(affectedComponent.getId());
|
||||||
|
if (state != null) {
|
||||||
|
affectedComponent.getComponent().setState(state);
|
||||||
|
}
|
||||||
|
} else {
|
||||||
|
affectedComponent.setPermissions(permissions);
|
||||||
|
affectedComponent.setComponent(null);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
|
@ -16,23 +16,7 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.nifi.groups;
|
package org.apache.nifi.groups;
|
||||||
|
|
||||||
import static java.util.Objects.requireNonNull;
|
import com.google.common.collect.Sets;
|
||||||
|
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.Collections;
|
|
||||||
import java.util.HashMap;
|
|
||||||
import java.util.HashSet;
|
|
||||||
import java.util.LinkedHashSet;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Map;
|
|
||||||
import java.util.Objects;
|
|
||||||
import java.util.Set;
|
|
||||||
import java.util.concurrent.CompletableFuture;
|
|
||||||
import java.util.concurrent.atomic.AtomicReference;
|
|
||||||
import java.util.concurrent.locks.Lock;
|
|
||||||
import java.util.concurrent.locks.ReentrantReadWriteLock;
|
|
||||||
import java.util.stream.Collectors;
|
|
||||||
|
|
||||||
import org.apache.commons.lang3.StringUtils;
|
import org.apache.commons.lang3.StringUtils;
|
||||||
import org.apache.commons.lang3.builder.HashCodeBuilder;
|
import org.apache.commons.lang3.builder.HashCodeBuilder;
|
||||||
import org.apache.commons.lang3.builder.ToStringBuilder;
|
import org.apache.commons.lang3.builder.ToStringBuilder;
|
||||||
|
@ -76,6 +60,7 @@ import org.apache.nifi.logging.LogRepositoryFactory;
|
||||||
import org.apache.nifi.nar.ExtensionManager;
|
import org.apache.nifi.nar.ExtensionManager;
|
||||||
import org.apache.nifi.nar.NarCloseable;
|
import org.apache.nifi.nar.NarCloseable;
|
||||||
import org.apache.nifi.processor.StandardProcessContext;
|
import org.apache.nifi.processor.StandardProcessContext;
|
||||||
|
import org.apache.nifi.registry.ComponentVariableRegistry;
|
||||||
import org.apache.nifi.registry.VariableDescriptor;
|
import org.apache.nifi.registry.VariableDescriptor;
|
||||||
import org.apache.nifi.registry.variable.MutableVariableRegistry;
|
import org.apache.nifi.registry.variable.MutableVariableRegistry;
|
||||||
import org.apache.nifi.remote.RemoteGroupPort;
|
import org.apache.nifi.remote.RemoteGroupPort;
|
||||||
|
@ -87,7 +72,22 @@ import org.apache.nifi.web.api.dto.TemplateDTO;
|
||||||
import org.slf4j.Logger;
|
import org.slf4j.Logger;
|
||||||
import org.slf4j.LoggerFactory;
|
import org.slf4j.LoggerFactory;
|
||||||
|
|
||||||
import com.google.common.collect.Sets;
|
import java.util.ArrayList;
|
||||||
|
import java.util.Collections;
|
||||||
|
import java.util.HashMap;
|
||||||
|
import java.util.HashSet;
|
||||||
|
import java.util.LinkedHashSet;
|
||||||
|
import java.util.List;
|
||||||
|
import java.util.Map;
|
||||||
|
import java.util.Objects;
|
||||||
|
import java.util.Set;
|
||||||
|
import java.util.concurrent.CompletableFuture;
|
||||||
|
import java.util.concurrent.atomic.AtomicReference;
|
||||||
|
import java.util.concurrent.locks.Lock;
|
||||||
|
import java.util.concurrent.locks.ReentrantReadWriteLock;
|
||||||
|
import java.util.stream.Collectors;
|
||||||
|
|
||||||
|
import static java.util.Objects.requireNonNull;
|
||||||
|
|
||||||
public final class StandardProcessGroup implements ProcessGroup {
|
public final class StandardProcessGroup implements ProcessGroup {
|
||||||
|
|
||||||
|
@ -2651,7 +2651,7 @@ public final class StandardProcessGroup implements ProcessGroup {
|
||||||
final Set<ConfiguredComponent> affected = new HashSet<>();
|
final Set<ConfiguredComponent> affected = new HashSet<>();
|
||||||
|
|
||||||
// Determine any Processors that references the variable
|
// Determine any Processors that references the variable
|
||||||
for (final ProcessorNode processor : findAllProcessors()) {
|
for (final ProcessorNode processor : getProcessors()) {
|
||||||
for (final VariableImpact impact : getVariableImpact(processor)) {
|
for (final VariableImpact impact : getVariableImpact(processor)) {
|
||||||
if (impact.isImpacted(variableName)) {
|
if (impact.isImpacted(variableName)) {
|
||||||
affected.add(processor);
|
affected.add(processor);
|
||||||
|
@ -2662,7 +2662,7 @@ public final class StandardProcessGroup implements ProcessGroup {
|
||||||
// Determine any Controller Service that references the variable. If Service A references a variable,
|
// Determine any Controller Service that references the variable. If Service A references a variable,
|
||||||
// then that means that any other component that references that service is also affected, so recursively
|
// then that means that any other component that references that service is also affected, so recursively
|
||||||
// find any references to that service and add it.
|
// find any references to that service and add it.
|
||||||
for (final ControllerServiceNode service : findAllControllerServices()) {
|
for (final ControllerServiceNode service : getControllerServices(false)) {
|
||||||
for (final VariableImpact impact : getVariableImpact(service)) {
|
for (final VariableImpact impact : getVariableImpact(service)) {
|
||||||
if (impact.isImpacted(variableName)) {
|
if (impact.isImpacted(variableName)) {
|
||||||
affected.add(service);
|
affected.add(service);
|
||||||
|
@ -2673,6 +2673,18 @@ public final class StandardProcessGroup implements ProcessGroup {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// For any child Process Group that does not override the variable, also include its references.
|
||||||
|
// If a child group has a value for the same variable, though, then that means that the child group
|
||||||
|
// is overriding the variable and its components are actually referencing a different variable.
|
||||||
|
for (final ProcessGroup childGroup : getProcessGroups()) {
|
||||||
|
final ComponentVariableRegistry childRegistry = childGroup.getVariableRegistry();
|
||||||
|
final VariableDescriptor descriptor = childRegistry.getVariableKey(variableName);
|
||||||
|
final boolean overridden = childRegistry.getVariableMap().containsKey(descriptor);
|
||||||
|
if (!overridden) {
|
||||||
|
affected.addAll(childGroup.getComponentsAffectedByVariable(variableName));
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
return affected;
|
return affected;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -2695,7 +2707,11 @@ public final class StandardProcessGroup implements ProcessGroup {
|
||||||
}
|
}
|
||||||
|
|
||||||
private List<VariableImpact> getVariableImpact(final ConfiguredComponent component) {
|
private List<VariableImpact> getVariableImpact(final ConfiguredComponent component) {
|
||||||
return component.getProperties().values().stream()
|
return component.getProperties().keySet().stream()
|
||||||
|
.map(descriptor -> {
|
||||||
|
final String configuredVal = component.getProperty(descriptor);
|
||||||
|
return configuredVal == null ? descriptor.getDefaultValue() : configuredVal;
|
||||||
|
})
|
||||||
.map(propVal -> Query.prepare(propVal).getVariableImpact())
|
.map(propVal -> Query.prepare(propVal).getVariableImpact())
|
||||||
.collect(Collectors.toList());
|
.collect(Collectors.toList());
|
||||||
}
|
}
|
||||||
|
|
|
@ -17,17 +17,28 @@
|
||||||
|
|
||||||
package org.apache.nifi.registry.variable;
|
package org.apache.nifi.registry.variable;
|
||||||
|
|
||||||
|
import org.apache.nifi.authorization.user.NiFiUser;
|
||||||
|
import org.apache.nifi.web.api.dto.RevisionDTO;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
|
|
||||||
import java.util.Date;
|
import java.util.Date;
|
||||||
|
import java.util.Map;
|
||||||
|
import java.util.Set;
|
||||||
import java.util.concurrent.atomic.AtomicReference;
|
import java.util.concurrent.atomic.AtomicReference;
|
||||||
|
import java.util.function.Function;
|
||||||
|
import java.util.stream.Collectors;
|
||||||
|
|
||||||
public class VariableRegistryUpdateRequest {
|
public class VariableRegistryUpdateRequest {
|
||||||
private final String requestId;
|
private final String requestId;
|
||||||
private final String processGroupId;
|
private final String processGroupId;
|
||||||
|
private final NiFiUser user;
|
||||||
private volatile Date submissionTime = new Date();
|
private volatile Date submissionTime = new Date();
|
||||||
private volatile Date lastUpdated = new Date();
|
private volatile Date lastUpdated = new Date();
|
||||||
private volatile boolean complete = false;
|
private volatile boolean complete = false;
|
||||||
|
|
||||||
private final AtomicReference<String> failureReason = new AtomicReference<>();
|
private final AtomicReference<String> failureReason = new AtomicReference<>();
|
||||||
|
private RevisionDTO processGroupRevision;
|
||||||
|
private Map<String, AffectedComponentEntity> affectedComponents;
|
||||||
|
|
||||||
private final VariableRegistryUpdateStep identifyComponentsStep = new VariableRegistryUpdateStep("Identifying components affected");
|
private final VariableRegistryUpdateStep identifyComponentsStep = new VariableRegistryUpdateStep("Identifying components affected");
|
||||||
private final VariableRegistryUpdateStep stopProcessors = new VariableRegistryUpdateStep("Stopping affected Processors");
|
private final VariableRegistryUpdateStep stopProcessors = new VariableRegistryUpdateStep("Stopping affected Processors");
|
||||||
|
@ -36,16 +47,17 @@ public class VariableRegistryUpdateRequest {
|
||||||
private final VariableRegistryUpdateStep enableServices = new VariableRegistryUpdateStep("Re-Enabling affected Controller Services");
|
private final VariableRegistryUpdateStep enableServices = new VariableRegistryUpdateStep("Re-Enabling affected Controller Services");
|
||||||
private final VariableRegistryUpdateStep startProcessors = new VariableRegistryUpdateStep("Restarting affected Processors");
|
private final VariableRegistryUpdateStep startProcessors = new VariableRegistryUpdateStep("Restarting affected Processors");
|
||||||
|
|
||||||
public VariableRegistryUpdateRequest(final String requestId, final String processGroupId) {
|
public VariableRegistryUpdateRequest(final String requestId, final String processGroupId, final Set<AffectedComponentEntity> affectedComponents, final NiFiUser user) {
|
||||||
this.requestId = requestId;
|
this.requestId = requestId;
|
||||||
this.processGroupId = processGroupId;
|
this.processGroupId = processGroupId;
|
||||||
|
this.affectedComponents = affectedComponents.stream().collect(Collectors.toMap(AffectedComponentEntity::getId, Function.identity()));
|
||||||
|
this.user = user;
|
||||||
}
|
}
|
||||||
|
|
||||||
public String getProcessGroupId() {
|
public String getProcessGroupId() {
|
||||||
return processGroupId;
|
return processGroupId;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
||||||
public String getRequestId() {
|
public String getRequestId() {
|
||||||
return requestId;
|
return requestId;
|
||||||
}
|
}
|
||||||
|
@ -54,6 +66,10 @@ public class VariableRegistryUpdateRequest {
|
||||||
return submissionTime;
|
return submissionTime;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public NiFiUser getUser() {
|
||||||
|
return user;
|
||||||
|
}
|
||||||
|
|
||||||
public Date getLastUpdated() {
|
public Date getLastUpdated() {
|
||||||
return lastUpdated;
|
return lastUpdated;
|
||||||
}
|
}
|
||||||
|
@ -102,6 +118,18 @@ public class VariableRegistryUpdateRequest {
|
||||||
this.failureReason.set(reason);
|
this.failureReason.set(reason);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public RevisionDTO getProcessGroupRevision() {
|
||||||
|
return processGroupRevision;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void setProcessGroupRevision(RevisionDTO processGroupRevision) {
|
||||||
|
this.processGroupRevision = processGroupRevision;
|
||||||
|
}
|
||||||
|
|
||||||
|
public Map<String, AffectedComponentEntity> getAffectedComponents() {
|
||||||
|
return affectedComponents;
|
||||||
|
}
|
||||||
|
|
||||||
public void cancel() {
|
public void cancel() {
|
||||||
this.failureReason.compareAndSet(null, "Update was cancelled");
|
this.failureReason.compareAndSet(null, "Update was cancelled");
|
||||||
this.complete = true;
|
this.complete = true;
|
||||||
|
|
|
@ -16,13 +16,6 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.nifi.web;
|
package org.apache.nifi.web;
|
||||||
|
|
||||||
import java.util.Date;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Map;
|
|
||||||
import java.util.Optional;
|
|
||||||
import java.util.Set;
|
|
||||||
import java.util.function.Function;
|
|
||||||
|
|
||||||
import org.apache.nifi.authorization.AuthorizeAccess;
|
import org.apache.nifi.authorization.AuthorizeAccess;
|
||||||
import org.apache.nifi.authorization.RequestAction;
|
import org.apache.nifi.authorization.RequestAction;
|
||||||
import org.apache.nifi.authorization.user.NiFiUser;
|
import org.apache.nifi.authorization.user.NiFiUser;
|
||||||
|
@ -77,6 +70,7 @@ import org.apache.nifi.web.api.dto.status.ControllerStatusDTO;
|
||||||
import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
||||||
import org.apache.nifi.web.api.entity.ActionEntity;
|
import org.apache.nifi.web.api.entity.ActionEntity;
|
||||||
import org.apache.nifi.web.api.entity.ActivateControllerServicesEntity;
|
import org.apache.nifi.web.api.entity.ActivateControllerServicesEntity;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
import org.apache.nifi.web.api.entity.BulletinEntity;
|
import org.apache.nifi.web.api.entity.BulletinEntity;
|
||||||
import org.apache.nifi.web.api.entity.ConnectionEntity;
|
import org.apache.nifi.web.api.entity.ConnectionEntity;
|
||||||
import org.apache.nifi.web.api.entity.ConnectionStatusEntity;
|
import org.apache.nifi.web.api.entity.ConnectionStatusEntity;
|
||||||
|
@ -108,6 +102,13 @@ import org.apache.nifi.web.api.entity.UserEntity;
|
||||||
import org.apache.nifi.web.api.entity.UserGroupEntity;
|
import org.apache.nifi.web.api.entity.UserGroupEntity;
|
||||||
import org.apache.nifi.web.api.entity.VariableRegistryEntity;
|
import org.apache.nifi.web.api.entity.VariableRegistryEntity;
|
||||||
|
|
||||||
|
import java.util.Date;
|
||||||
|
import java.util.List;
|
||||||
|
import java.util.Map;
|
||||||
|
import java.util.Optional;
|
||||||
|
import java.util.Set;
|
||||||
|
import java.util.function.Function;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Defines the NiFiServiceFacade interface.
|
* Defines the NiFiServiceFacade interface.
|
||||||
*/
|
*/
|
||||||
|
@ -518,9 +519,10 @@ public interface NiFiServiceFacade {
|
||||||
* Gets all the Processor transfer objects for this controller.
|
* Gets all the Processor transfer objects for this controller.
|
||||||
*
|
*
|
||||||
* @param groupId group
|
* @param groupId group
|
||||||
|
* @param includeDescendants if processors from descendent groups should be included
|
||||||
* @return List of all the Processor transfer object
|
* @return List of all the Processor transfer object
|
||||||
*/
|
*/
|
||||||
Set<ProcessorEntity> getProcessors(String groupId);
|
Set<ProcessorEntity> getProcessors(String groupId, boolean includeDescendants);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Verifies the specified processor can be updated.
|
* Verifies the specified processor can be updated.
|
||||||
|
@ -925,12 +927,21 @@ public interface NiFiServiceFacade {
|
||||||
VariableRegistryEntity updateVariableRegistry(NiFiUser user, Revision revision, VariableRegistryDTO variableRegistryDto);
|
VariableRegistryEntity updateVariableRegistry(NiFiUser user, Revision revision, VariableRegistryDTO variableRegistryDto);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Determines which components will be affected by updating the given Variable Registry
|
* Determines which components will be affected by updating the given Variable Registry.
|
||||||
*
|
*
|
||||||
* @param variableRegistryDto the variable registry
|
* @param variableRegistryDto the variable registry
|
||||||
* @return the components that will be affected
|
* @return the components that will be affected
|
||||||
*/
|
*/
|
||||||
Set<AffectedComponentDTO> getComponentsAffectedByVariableRegistryUpdate(VariableRegistryDTO variableRegistryDto);
|
Set<AffectedComponentEntity> getComponentsAffectedByVariableRegistryUpdate(VariableRegistryDTO variableRegistryDto);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Determines which components are active and will be affected by updating the given Variable Registry. These active components
|
||||||
|
* are needed to authorize the request and deactivate prior to changing the variables.
|
||||||
|
*
|
||||||
|
* @param variableRegistryDto the variable registry
|
||||||
|
* @return the components that will be affected
|
||||||
|
*/
|
||||||
|
Set<AffectedComponentDTO> getActiveComponentsAffectedByVariableRegistryUpdate(VariableRegistryDTO variableRegistryDto);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Gets all process groups in the specified parent group.
|
* Gets all process groups in the specified parent group.
|
||||||
|
|
|
@ -161,6 +161,7 @@ import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
||||||
import org.apache.nifi.web.api.entity.AccessPolicySummaryEntity;
|
import org.apache.nifi.web.api.entity.AccessPolicySummaryEntity;
|
||||||
import org.apache.nifi.web.api.entity.ActionEntity;
|
import org.apache.nifi.web.api.entity.ActionEntity;
|
||||||
import org.apache.nifi.web.api.entity.ActivateControllerServicesEntity;
|
import org.apache.nifi.web.api.entity.ActivateControllerServicesEntity;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
import org.apache.nifi.web.api.entity.BulletinEntity;
|
import org.apache.nifi.web.api.entity.BulletinEntity;
|
||||||
import org.apache.nifi.web.api.entity.ComponentReferenceEntity;
|
import org.apache.nifi.web.api.entity.ComponentReferenceEntity;
|
||||||
import org.apache.nifi.web.api.entity.ConnectionEntity;
|
import org.apache.nifi.web.api.entity.ConnectionEntity;
|
||||||
|
@ -790,7 +791,7 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public Set<AffectedComponentDTO> getComponentsAffectedByVariableRegistryUpdate(final VariableRegistryDTO variableRegistryDto) {
|
public Set<AffectedComponentDTO> getActiveComponentsAffectedByVariableRegistryUpdate(final VariableRegistryDTO variableRegistryDto) {
|
||||||
final ProcessGroup group = processGroupDAO.getProcessGroup(variableRegistryDto.getProcessGroupId());
|
final ProcessGroup group = processGroupDAO.getProcessGroup(variableRegistryDto.getProcessGroupId());
|
||||||
if (group == null) {
|
if (group == null) {
|
||||||
throw new ResourceNotFoundException("Could not find Process Group with ID " + variableRegistryDto.getProcessGroupId());
|
throw new ResourceNotFoundException("Could not find Process Group with ID " + variableRegistryDto.getProcessGroupId());
|
||||||
|
@ -827,6 +828,29 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
return affectedComponentDtos;
|
return affectedComponentDtos;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
public Set<AffectedComponentEntity> getComponentsAffectedByVariableRegistryUpdate(final VariableRegistryDTO variableRegistryDto) {
|
||||||
|
final ProcessGroup group = processGroupDAO.getProcessGroup(variableRegistryDto.getProcessGroupId());
|
||||||
|
if (group == null) {
|
||||||
|
throw new ResourceNotFoundException("Could not find Process Group with ID " + variableRegistryDto.getProcessGroupId());
|
||||||
|
}
|
||||||
|
|
||||||
|
final Map<String, String> variableMap = new HashMap<>();
|
||||||
|
variableRegistryDto.getVariables().stream() // have to use forEach here instead of using Collectors.toMap because value may be null
|
||||||
|
.map(VariableEntity::getVariable)
|
||||||
|
.forEach(var -> variableMap.put(var.getName(), var.getValue()));
|
||||||
|
|
||||||
|
final Set<AffectedComponentEntity> affectedComponentEntities = new HashSet<>();
|
||||||
|
|
||||||
|
final Set<String> updatedVariableNames = getUpdatedVariables(group, variableMap);
|
||||||
|
for (final String variableName : updatedVariableNames) {
|
||||||
|
final Set<ConfiguredComponent> affectedComponents = group.getComponentsAffectedByVariable(variableName);
|
||||||
|
affectedComponentEntities.addAll(dtoFactory.createAffectedComponentEntities(affectedComponents, revisionManager));
|
||||||
|
}
|
||||||
|
|
||||||
|
return affectedComponentEntities;
|
||||||
|
}
|
||||||
|
|
||||||
private Set<String> getUpdatedVariables(final ProcessGroup group, final Map<String, String> newVariableValues) {
|
private Set<String> getUpdatedVariables(final ProcessGroup group, final Map<String, String> newVariableValues) {
|
||||||
final Set<String> updatedVariableNames = new HashSet<>();
|
final Set<String> updatedVariableNames = new HashSet<>();
|
||||||
|
|
||||||
|
@ -856,7 +880,7 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
final RevisionUpdate<VariableRegistryDTO> snapshot = updateComponent(user, revision,
|
final RevisionUpdate<VariableRegistryDTO> snapshot = updateComponent(user, revision,
|
||||||
processGroupNode,
|
processGroupNode,
|
||||||
() -> processGroupDAO.updateVariableRegistry(variableRegistryDto),
|
() -> processGroupDAO.updateVariableRegistry(variableRegistryDto),
|
||||||
processGroup -> dtoFactory.createVariableRegistryDto(processGroup));
|
processGroup -> dtoFactory.createVariableRegistryDto(processGroup, revisionManager));
|
||||||
|
|
||||||
final PermissionsDTO permissions = dtoFactory.createPermissionsDto(processGroupNode);
|
final PermissionsDTO permissions = dtoFactory.createPermissionsDto(processGroupNode);
|
||||||
final RevisionDTO updatedRevision = dtoFactory.createRevisionDTO(snapshot.getLastModification());
|
final RevisionDTO updatedRevision = dtoFactory.createRevisionDTO(snapshot.getLastModification());
|
||||||
|
@ -2498,8 +2522,8 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public Set<ProcessorEntity> getProcessors(final String groupId) {
|
public Set<ProcessorEntity> getProcessors(final String groupId, final boolean includeDescendants) {
|
||||||
final Set<ProcessorNode> processors = processorDAO.getProcessors(groupId);
|
final Set<ProcessorNode> processors = processorDAO.getProcessors(groupId, includeDescendants);
|
||||||
return processors.stream()
|
return processors.stream()
|
||||||
.map(processor -> createProcessorEntity(processor))
|
.map(processor -> createProcessorEntity(processor))
|
||||||
.collect(Collectors.toSet());
|
.collect(Collectors.toSet());
|
||||||
|
@ -3231,7 +3255,7 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
}
|
}
|
||||||
|
|
||||||
private VariableRegistryEntity createVariableRegistryEntity(final ProcessGroup processGroup, final boolean includeAncestorGroups) {
|
private VariableRegistryEntity createVariableRegistryEntity(final ProcessGroup processGroup, final boolean includeAncestorGroups) {
|
||||||
final VariableRegistryDTO registryDto = dtoFactory.createVariableRegistryDto(processGroup);
|
final VariableRegistryDTO registryDto = dtoFactory.createVariableRegistryDto(processGroup, revisionManager);
|
||||||
final RevisionDTO revision = dtoFactory.createRevisionDTO(revisionManager.getRevision(processGroup.getIdentifier()));
|
final RevisionDTO revision = dtoFactory.createRevisionDTO(revisionManager.getRevision(processGroup.getIdentifier()));
|
||||||
final PermissionsDTO permissions = dtoFactory.createPermissionsDto(processGroup);
|
final PermissionsDTO permissions = dtoFactory.createPermissionsDto(processGroup);
|
||||||
|
|
||||||
|
@ -3240,7 +3264,7 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
while (parent != null) {
|
while (parent != null) {
|
||||||
final PermissionsDTO parentPerms = dtoFactory.createPermissionsDto(parent);
|
final PermissionsDTO parentPerms = dtoFactory.createPermissionsDto(parent);
|
||||||
if (Boolean.TRUE.equals(parentPerms.getCanRead())) {
|
if (Boolean.TRUE.equals(parentPerms.getCanRead())) {
|
||||||
final VariableRegistryDTO parentRegistryDto = dtoFactory.createVariableRegistryDto(parent);
|
final VariableRegistryDTO parentRegistryDto = dtoFactory.createVariableRegistryDto(parent, revisionManager);
|
||||||
final Set<VariableEntity> parentVariables = parentRegistryDto.getVariables();
|
final Set<VariableEntity> parentVariables = parentRegistryDto.getVariables();
|
||||||
registryDto.getVariables().addAll(parentVariables);
|
registryDto.getVariables().addAll(parentVariables);
|
||||||
}
|
}
|
||||||
|
@ -3260,7 +3284,7 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
|
||||||
throw new ResourceNotFoundException("Could not find group with ID " + groupId);
|
throw new ResourceNotFoundException("Could not find group with ID " + groupId);
|
||||||
}
|
}
|
||||||
|
|
||||||
final VariableRegistryDTO registryDto = dtoFactory.populateAffectedComponents(variableRegistryDto, processGroup);
|
final VariableRegistryDTO registryDto = dtoFactory.populateAffectedComponents(variableRegistryDto, processGroup, revisionManager);
|
||||||
final RevisionDTO revision = dtoFactory.createRevisionDTO(revisionManager.getRevision(processGroup.getIdentifier()));
|
final RevisionDTO revision = dtoFactory.createRevisionDTO(revisionManager.getRevision(processGroup.getIdentifier()));
|
||||||
final PermissionsDTO permissions = dtoFactory.createPermissionsDto(processGroup);
|
final PermissionsDTO permissions = dtoFactory.createPermissionsDto(processGroup);
|
||||||
return entityFactory.createVariableRegistryEntity(registryDto, revision, permissions);
|
return entityFactory.createVariableRegistryEntity(registryDto, revision, permissions);
|
||||||
|
|
File diff suppressed because it is too large
Load Diff
|
@ -16,33 +16,6 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.nifi.web.api.dto;
|
package org.apache.nifi.web.api.dto;
|
||||||
|
|
||||||
import java.text.Collator;
|
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.Arrays;
|
|
||||||
import java.util.Collection;
|
|
||||||
import java.util.Collections;
|
|
||||||
import java.util.Comparator;
|
|
||||||
import java.util.Date;
|
|
||||||
import java.util.HashMap;
|
|
||||||
import java.util.HashSet;
|
|
||||||
import java.util.Iterator;
|
|
||||||
import java.util.LinkedHashMap;
|
|
||||||
import java.util.LinkedHashSet;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Locale;
|
|
||||||
import java.util.Map;
|
|
||||||
import java.util.Map.Entry;
|
|
||||||
import java.util.Set;
|
|
||||||
import java.util.TimeZone;
|
|
||||||
import java.util.TreeMap;
|
|
||||||
import java.util.TreeSet;
|
|
||||||
import java.util.concurrent.TimeUnit;
|
|
||||||
import java.util.function.Function;
|
|
||||||
import java.util.function.Supplier;
|
|
||||||
import java.util.stream.Collectors;
|
|
||||||
|
|
||||||
import javax.ws.rs.WebApplicationException;
|
|
||||||
|
|
||||||
import org.apache.commons.lang3.ClassUtils;
|
import org.apache.commons.lang3.ClassUtils;
|
||||||
import org.apache.commons.lang3.StringUtils;
|
import org.apache.commons.lang3.StringUtils;
|
||||||
import org.apache.nifi.action.Action;
|
import org.apache.nifi.action.Action;
|
||||||
|
@ -182,6 +155,7 @@ import org.apache.nifi.web.api.dto.status.RemoteProcessGroupStatusDTO;
|
||||||
import org.apache.nifi.web.api.dto.status.RemoteProcessGroupStatusSnapshotDTO;
|
import org.apache.nifi.web.api.dto.status.RemoteProcessGroupStatusSnapshotDTO;
|
||||||
import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
||||||
import org.apache.nifi.web.api.entity.AccessPolicySummaryEntity;
|
import org.apache.nifi.web.api.entity.AccessPolicySummaryEntity;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
import org.apache.nifi.web.api.entity.AllowableValueEntity;
|
import org.apache.nifi.web.api.entity.AllowableValueEntity;
|
||||||
import org.apache.nifi.web.api.entity.BulletinEntity;
|
import org.apache.nifi.web.api.entity.BulletinEntity;
|
||||||
import org.apache.nifi.web.api.entity.ComponentReferenceEntity;
|
import org.apache.nifi.web.api.entity.ComponentReferenceEntity;
|
||||||
|
@ -196,6 +170,32 @@ import org.apache.nifi.web.api.entity.VariableEntity;
|
||||||
import org.apache.nifi.web.controller.ControllerFacade;
|
import org.apache.nifi.web.controller.ControllerFacade;
|
||||||
import org.apache.nifi.web.revision.RevisionManager;
|
import org.apache.nifi.web.revision.RevisionManager;
|
||||||
|
|
||||||
|
import javax.ws.rs.WebApplicationException;
|
||||||
|
import java.text.Collator;
|
||||||
|
import java.util.ArrayList;
|
||||||
|
import java.util.Arrays;
|
||||||
|
import java.util.Collection;
|
||||||
|
import java.util.Collections;
|
||||||
|
import java.util.Comparator;
|
||||||
|
import java.util.Date;
|
||||||
|
import java.util.HashMap;
|
||||||
|
import java.util.HashSet;
|
||||||
|
import java.util.Iterator;
|
||||||
|
import java.util.LinkedHashMap;
|
||||||
|
import java.util.LinkedHashSet;
|
||||||
|
import java.util.List;
|
||||||
|
import java.util.Locale;
|
||||||
|
import java.util.Map;
|
||||||
|
import java.util.Map.Entry;
|
||||||
|
import java.util.Set;
|
||||||
|
import java.util.TimeZone;
|
||||||
|
import java.util.TreeMap;
|
||||||
|
import java.util.TreeSet;
|
||||||
|
import java.util.concurrent.TimeUnit;
|
||||||
|
import java.util.function.Function;
|
||||||
|
import java.util.function.Supplier;
|
||||||
|
import java.util.stream.Collectors;
|
||||||
|
|
||||||
public final class DtoFactory {
|
public final class DtoFactory {
|
||||||
|
|
||||||
@SuppressWarnings("rawtypes")
|
@SuppressWarnings("rawtypes")
|
||||||
|
@ -1737,13 +1737,29 @@ public final class DtoFactory {
|
||||||
|
|
||||||
public AffectedComponentDTO createAffectedComponentDto(final ConfiguredComponent component) {
|
public AffectedComponentDTO createAffectedComponentDto(final ConfiguredComponent component) {
|
||||||
final AffectedComponentDTO dto = new AffectedComponentDTO();
|
final AffectedComponentDTO dto = new AffectedComponentDTO();
|
||||||
dto.setComponentId(component.getIdentifier());
|
dto.setId(component.getIdentifier());
|
||||||
dto.setParentGroupId(component.getProcessGroupIdentifier());
|
dto.setName(component.getName());
|
||||||
|
dto.setProcessGroupId(component.getProcessGroupIdentifier());
|
||||||
|
|
||||||
if (component instanceof ProcessorNode) {
|
if (component instanceof ProcessorNode) {
|
||||||
dto.setComponentType(AffectedComponentDTO.COMPONENT_TYPE_PROCESSOR);
|
final ProcessorNode node = ((ProcessorNode) component);
|
||||||
|
dto.setState(node.getScheduledState().name());
|
||||||
|
dto.setActiveThreadCount(node.getActiveThreadCount());
|
||||||
|
dto.setReferenceType(AffectedComponentDTO.COMPONENT_TYPE_PROCESSOR);
|
||||||
} else if (component instanceof ControllerServiceNode) {
|
} else if (component instanceof ControllerServiceNode) {
|
||||||
dto.setComponentType(AffectedComponentDTO.COMPONENT_TYPE_CONTROLLER_SERVICE);
|
final ControllerServiceNode node = ((ControllerServiceNode) component);
|
||||||
|
dto.setState(node.getState().name());
|
||||||
|
dto.setReferenceType(AffectedComponentDTO.COMPONENT_TYPE_CONTROLLER_SERVICE);
|
||||||
|
}
|
||||||
|
|
||||||
|
final Collection<ValidationResult> validationErrors = component.getValidationErrors();
|
||||||
|
if (validationErrors != null && !validationErrors.isEmpty()) {
|
||||||
|
final List<String> errors = new ArrayList<>();
|
||||||
|
for (final ValidationResult validationResult : validationErrors) {
|
||||||
|
errors.add(validationResult.toString());
|
||||||
|
}
|
||||||
|
|
||||||
|
dto.setValidationErrors(errors);
|
||||||
}
|
}
|
||||||
|
|
||||||
return dto;
|
return dto;
|
||||||
|
@ -2114,8 +2130,18 @@ public final class DtoFactory {
|
||||||
return deprecationNotice == null ? null : deprecationNotice.reason();
|
return deprecationNotice == null ? null : deprecationNotice.reason();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public Set<AffectedComponentEntity> createAffectedComponentEntities(final Set<ConfiguredComponent> affectedComponents, final RevisionManager revisionManager) {
|
||||||
|
return affectedComponents.stream()
|
||||||
|
.map(component -> {
|
||||||
|
final AffectedComponentDTO affectedComponent = createAffectedComponentDto(component);
|
||||||
|
final PermissionsDTO permissions = createPermissionsDto(component);
|
||||||
|
final RevisionDTO revision = createRevisionDTO(revisionManager.getRevision(component.getIdentifier()));
|
||||||
|
return entityFactory.createAffectedComponentEntity(affectedComponent, revision, permissions);
|
||||||
|
})
|
||||||
|
.collect(Collectors.toSet());
|
||||||
|
}
|
||||||
|
|
||||||
public VariableRegistryDTO createVariableRegistryDto(final ProcessGroup processGroup) {
|
public VariableRegistryDTO createVariableRegistryDto(final ProcessGroup processGroup, final RevisionManager revisionManager) {
|
||||||
final ComponentVariableRegistry variableRegistry = processGroup.getVariableRegistry();
|
final ComponentVariableRegistry variableRegistry = processGroup.getVariableRegistry();
|
||||||
|
|
||||||
final List<String> variableNames = variableRegistry.getVariableMap().keySet().stream()
|
final List<String> variableNames = variableRegistry.getVariableMap().keySet().stream()
|
||||||
|
@ -2130,21 +2156,18 @@ public final class DtoFactory {
|
||||||
variableDto.setValue(variableRegistry.getVariableValue(variableName));
|
variableDto.setValue(variableRegistry.getVariableValue(variableName));
|
||||||
variableDto.setProcessGroupId(processGroup.getIdentifier());
|
variableDto.setProcessGroupId(processGroup.getIdentifier());
|
||||||
|
|
||||||
final Set<ConfiguredComponent> affectedComponents = processGroup.getComponentsAffectedByVariable(variableName);
|
final Set<AffectedComponentEntity> affectedComponentEntities = createAffectedComponentEntities(processGroup.getComponentsAffectedByVariable(variableName), revisionManager);
|
||||||
final Set<AffectedComponentDTO> affectedComponentDtos = affectedComponents.stream()
|
|
||||||
.map(component -> createAffectedComponentDto(component))
|
|
||||||
.collect(Collectors.toSet());
|
|
||||||
|
|
||||||
boolean canWrite = true;
|
boolean canWrite = true;
|
||||||
for (final ConfiguredComponent component : affectedComponents) {
|
for (final AffectedComponentEntity affectedComponent : affectedComponentEntities) {
|
||||||
final PermissionsDTO permissions = createPermissionsDto(component);
|
final PermissionsDTO permissions = affectedComponent.getPermissions();
|
||||||
if (!permissions.getCanRead() || !permissions.getCanWrite()) {
|
if (!permissions.getCanRead() || !permissions.getCanWrite()) {
|
||||||
canWrite = false;
|
canWrite = false;
|
||||||
break;
|
break;
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
variableDto.setAffectedComponents(affectedComponentDtos);
|
variableDto.setAffectedComponents(affectedComponentEntities);
|
||||||
|
|
||||||
final VariableEntity variableEntity = new VariableEntity();
|
final VariableEntity variableEntity = new VariableEntity();
|
||||||
variableEntity.setVariable(variableDto);
|
variableEntity.setVariable(variableDto);
|
||||||
|
@ -2178,6 +2201,8 @@ public final class DtoFactory {
|
||||||
updateSteps.add(createVariableRegistryUpdateStepDto(request.getStartProcessorsStep()));
|
updateSteps.add(createVariableRegistryUpdateStepDto(request.getStartProcessorsStep()));
|
||||||
dto.setUpdateSteps(updateSteps);
|
dto.setUpdateSteps(updateSteps);
|
||||||
|
|
||||||
|
dto.setAffectedComponents(new HashSet<>(request.getAffectedComponents().values()));
|
||||||
|
|
||||||
return dto;
|
return dto;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -2190,42 +2215,41 @@ public final class DtoFactory {
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
||||||
public VariableRegistryDTO populateAffectedComponents(final VariableRegistryDTO variableRegistry, final ProcessGroup group) {
|
public VariableRegistryDTO populateAffectedComponents(final VariableRegistryDTO variableRegistry, final ProcessGroup group, final RevisionManager revisionManager) {
|
||||||
if (!group.getIdentifier().equals(variableRegistry.getProcessGroupId())) {
|
if (!group.getIdentifier().equals(variableRegistry.getProcessGroupId())) {
|
||||||
throw new IllegalArgumentException("Variable Registry does not have the same Group ID as the given Process Group");
|
throw new IllegalArgumentException("Variable Registry does not have the same Group ID as the given Process Group");
|
||||||
}
|
}
|
||||||
|
|
||||||
final Set<VariableEntity> variableEntities = new LinkedHashSet<>();
|
final Set<VariableEntity> variableEntities = new LinkedHashSet<>();
|
||||||
|
|
||||||
for (final VariableEntity inputEntity : variableRegistry.getVariables()) {
|
if (variableRegistry.getVariables() != null) {
|
||||||
final VariableEntity entity = new VariableEntity();
|
for (final VariableEntity inputEntity : variableRegistry.getVariables()) {
|
||||||
|
final VariableEntity entity = new VariableEntity();
|
||||||
|
|
||||||
final VariableDTO inputDto = inputEntity.getVariable();
|
final VariableDTO inputDto = inputEntity.getVariable();
|
||||||
final VariableDTO variableDto = new VariableDTO();
|
final VariableDTO variableDto = new VariableDTO();
|
||||||
variableDto.setName(inputDto.getName());
|
variableDto.setName(inputDto.getName());
|
||||||
variableDto.setValue(inputDto.getValue());
|
variableDto.setValue(inputDto.getValue());
|
||||||
variableDto.setProcessGroupId(group.getIdentifier());
|
variableDto.setProcessGroupId(group.getIdentifier());
|
||||||
|
|
||||||
final Set<ConfiguredComponent> affectedComponents = group.getComponentsAffectedByVariable(variableDto.getName());
|
final Set<AffectedComponentEntity> affectedComponentEntities = createAffectedComponentEntities(group.getComponentsAffectedByVariable(variableDto.getName()), revisionManager);
|
||||||
final Set<AffectedComponentDTO> affectedComponentDtos = affectedComponents.stream()
|
|
||||||
.map(component -> createAffectedComponentDto(component))
|
|
||||||
.collect(Collectors.toSet());
|
|
||||||
|
|
||||||
boolean canWrite = true;
|
boolean canWrite = true;
|
||||||
for (final ConfiguredComponent component : affectedComponents) {
|
for (final AffectedComponentEntity affectedComponent : affectedComponentEntities) {
|
||||||
final PermissionsDTO permissions = createPermissionsDto(component);
|
final PermissionsDTO permissions = affectedComponent.getPermissions();
|
||||||
if (!permissions.getCanRead() || !permissions.getCanWrite()) {
|
if (!permissions.getCanRead() || !permissions.getCanWrite()) {
|
||||||
canWrite = false;
|
canWrite = false;
|
||||||
break;
|
break;
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
variableDto.setAffectedComponents(affectedComponentEntities);
|
||||||
|
|
||||||
|
entity.setCanWrite(canWrite);
|
||||||
|
entity.setVariable(inputDto);
|
||||||
|
|
||||||
|
variableEntities.add(entity);
|
||||||
}
|
}
|
||||||
|
|
||||||
variableDto.setAffectedComponents(affectedComponentDtos);
|
|
||||||
|
|
||||||
entity.setCanWrite(canWrite);
|
|
||||||
entity.setVariable(inputDto);
|
|
||||||
|
|
||||||
variableEntities.add(entity);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
final VariableRegistryDTO registryDto = new VariableRegistryDTO();
|
final VariableRegistryDTO registryDto = new VariableRegistryDTO();
|
||||||
|
|
|
@ -33,6 +33,7 @@ import org.apache.nifi.web.api.dto.status.StatusHistoryDTO;
|
||||||
import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
import org.apache.nifi.web.api.entity.AccessPolicyEntity;
|
||||||
import org.apache.nifi.web.api.entity.AccessPolicySummaryEntity;
|
import org.apache.nifi.web.api.entity.AccessPolicySummaryEntity;
|
||||||
import org.apache.nifi.web.api.entity.ActionEntity;
|
import org.apache.nifi.web.api.entity.ActionEntity;
|
||||||
|
import org.apache.nifi.web.api.entity.AffectedComponentEntity;
|
||||||
import org.apache.nifi.web.api.entity.AllowableValueEntity;
|
import org.apache.nifi.web.api.entity.AllowableValueEntity;
|
||||||
import org.apache.nifi.web.api.entity.BulletinEntity;
|
import org.apache.nifi.web.api.entity.BulletinEntity;
|
||||||
import org.apache.nifi.web.api.entity.ComponentReferenceEntity;
|
import org.apache.nifi.web.api.entity.ComponentReferenceEntity;
|
||||||
|
@ -311,6 +312,20 @@ public final class EntityFactory {
|
||||||
return entity;
|
return entity;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public AffectedComponentEntity createAffectedComponentEntity(final AffectedComponentDTO dto, final RevisionDTO revision, final PermissionsDTO permissions) {
|
||||||
|
final AffectedComponentEntity entity = new AffectedComponentEntity();
|
||||||
|
entity.setRevision(revision);
|
||||||
|
if (dto != null) {
|
||||||
|
entity.setPermissions(permissions);
|
||||||
|
entity.setId(dto.getId());
|
||||||
|
|
||||||
|
if (permissions != null && permissions.getCanRead()) {
|
||||||
|
entity.setComponent(dto);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
return entity;
|
||||||
|
}
|
||||||
|
|
||||||
public UserGroupEntity createUserGroupEntity(final UserGroupDTO dto, final RevisionDTO revision, final PermissionsDTO permissions) {
|
public UserGroupEntity createUserGroupEntity(final UserGroupDTO dto, final RevisionDTO revision, final PermissionsDTO permissions) {
|
||||||
final UserGroupEntity entity = new UserGroupEntity();
|
final UserGroupEntity entity = new UserGroupEntity();
|
||||||
entity.setRevision(revision);
|
entity.setRevision(revision);
|
||||||
|
|
|
@ -59,9 +59,10 @@ public interface ProcessorDAO {
|
||||||
* Gets all the Processor transfer objects for this controller.
|
* Gets all the Processor transfer objects for this controller.
|
||||||
*
|
*
|
||||||
* @param groupId group id
|
* @param groupId group id
|
||||||
|
* @param includeDescendants if processors from descendant groups should be included
|
||||||
* @return List of all the Processors
|
* @return List of all the Processors
|
||||||
*/
|
*/
|
||||||
Set<ProcessorNode> getProcessors(String groupId);
|
Set<ProcessorNode> getProcessors(String groupId, boolean includeDescendants);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Verifies the specified processor can be updated.
|
* Verifies the specified processor can be updated.
|
||||||
|
|
|
@ -58,6 +58,7 @@ import java.util.Set;
|
||||||
import java.util.concurrent.RejectedExecutionException;
|
import java.util.concurrent.RejectedExecutionException;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
import java.util.regex.Matcher;
|
import java.util.regex.Matcher;
|
||||||
|
import java.util.stream.Collectors;
|
||||||
|
|
||||||
public class StandardProcessorDAO extends ComponentDAO implements ProcessorDAO {
|
public class StandardProcessorDAO extends ComponentDAO implements ProcessorDAO {
|
||||||
|
|
||||||
|
@ -307,9 +308,13 @@ public class StandardProcessorDAO extends ComponentDAO implements ProcessorDAO {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public Set<ProcessorNode> getProcessors(String groupId) {
|
public Set<ProcessorNode> getProcessors(String groupId, boolean includeDescendants) {
|
||||||
ProcessGroup group = locateProcessGroup(flowController, groupId);
|
ProcessGroup group = locateProcessGroup(flowController, groupId);
|
||||||
return group.getProcessors();
|
if (includeDescendants) {
|
||||||
|
return group.findAllProcessors().stream().collect(Collectors.toSet());
|
||||||
|
} else {
|
||||||
|
return group.getProcessors();
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
|
@ -461,6 +461,7 @@
|
||||||
<include>${staging.dir}/js/nf/canvas/nf-port-configuration.js</include>
|
<include>${staging.dir}/js/nf/canvas/nf-port-configuration.js</include>
|
||||||
<include>${staging.dir}/js/nf/canvas/nf-port-details.js</include>
|
<include>${staging.dir}/js/nf/canvas/nf-port-details.js</include>
|
||||||
<include>${staging.dir}/js/nf/canvas/nf-process-group-configuration.js</include>
|
<include>${staging.dir}/js/nf/canvas/nf-process-group-configuration.js</include>
|
||||||
|
<include>${staging.dir}/js/nf/canvas/nf-variable-registry.js</include>
|
||||||
<include>${staging.dir}/js/nf/canvas/nf-component-version.js</include>
|
<include>${staging.dir}/js/nf/canvas/nf-component-version.js</include>
|
||||||
<include>${staging.dir}/js/nf/canvas/nf-remote-process-group-configuration.js</include>
|
<include>${staging.dir}/js/nf/canvas/nf-remote-process-group-configuration.js</include>
|
||||||
<include>${staging.dir}/js/nf/canvas/nf-remote-process-group-details.js</include>
|
<include>${staging.dir}/js/nf/canvas/nf-remote-process-group-details.js</include>
|
||||||
|
|
|
@ -43,6 +43,7 @@ nf.canvas.script.tags=<script type="text/javascript" src="js/nf/nf-ng-bridge.js?
|
||||||
<script type="text/javascript" src="js/nf/canvas/nf-port-configuration.js?${project.version}"></script>\n\
|
<script type="text/javascript" src="js/nf/canvas/nf-port-configuration.js?${project.version}"></script>\n\
|
||||||
<script type="text/javascript" src="js/nf/canvas/nf-port-details.js?${project.version}"></script>\n\
|
<script type="text/javascript" src="js/nf/canvas/nf-port-details.js?${project.version}"></script>\n\
|
||||||
<script type="text/javascript" src="js/nf/canvas/nf-process-group-configuration.js?${project.version}"></script>\n\
|
<script type="text/javascript" src="js/nf/canvas/nf-process-group-configuration.js?${project.version}"></script>\n\
|
||||||
|
<script type="text/javascript" src="js/nf/canvas/nf-variable-registry.js?${project.version}"></script>\n\
|
||||||
<script type="text/javascript" src="js/nf/canvas/nf-component-version.js?${project.version}"></script>\n\
|
<script type="text/javascript" src="js/nf/canvas/nf-component-version.js?${project.version}"></script>\n\
|
||||||
<script type="text/javascript" src="js/nf/canvas/nf-remote-process-group-configuration.js?${project.version}"></script>\n\
|
<script type="text/javascript" src="js/nf/canvas/nf-remote-process-group-configuration.js?${project.version}"></script>\n\
|
||||||
<script type="text/javascript" src="js/nf/canvas/nf-remote-process-group-details.js?${project.version}"></script>\n\
|
<script type="text/javascript" src="js/nf/canvas/nf-remote-process-group-details.js?${project.version}"></script>\n\
|
||||||
|
|
|
@ -129,6 +129,7 @@
|
||||||
<jsp:include page="/WEB-INF/partials/canvas/reporting-task-configuration.jsp"/>
|
<jsp:include page="/WEB-INF/partials/canvas/reporting-task-configuration.jsp"/>
|
||||||
<jsp:include page="/WEB-INF/partials/canvas/processor-configuration.jsp"/>
|
<jsp:include page="/WEB-INF/partials/canvas/processor-configuration.jsp"/>
|
||||||
<jsp:include page="/WEB-INF/partials/processor-details.jsp"/>
|
<jsp:include page="/WEB-INF/partials/processor-details.jsp"/>
|
||||||
|
<jsp:include page="/WEB-INF/partials/canvas/variable-configuration.jsp"/>
|
||||||
<jsp:include page="/WEB-INF/partials/canvas/process-group-configuration.jsp"/>
|
<jsp:include page="/WEB-INF/partials/canvas/process-group-configuration.jsp"/>
|
||||||
<jsp:include page="/WEB-INF/partials/canvas/override-policy-dialog.jsp"/>
|
<jsp:include page="/WEB-INF/partials/canvas/override-policy-dialog.jsp"/>
|
||||||
<jsp:include page="/WEB-INF/partials/canvas/policy-management.jsp"/>
|
<jsp:include page="/WEB-INF/partials/canvas/policy-management.jsp"/>
|
||||||
|
|
|
@ -0,0 +1,93 @@
|
||||||
|
<%--
|
||||||
|
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.
|
||||||
|
--%>
|
||||||
|
<%@ page contentType="text/html" pageEncoding="UTF-8" session="false" %>
|
||||||
|
<div id="variable-registry-dialog" class="hidden">
|
||||||
|
<div class="dialog-content">
|
||||||
|
<div class="settings-left">
|
||||||
|
<div class="setting">
|
||||||
|
<div style="float: left;">
|
||||||
|
<div class="setting-name">Process Group</div>
|
||||||
|
<div class="setting-field">
|
||||||
|
<span id="process-group-variable-registry"></span>
|
||||||
|
<span id="variable-registry-process-group-id" class="hidden"></span>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
<div id="add-variable"><button class="button fa fa-plus"></button></div>
|
||||||
|
<div class="clear"></div>
|
||||||
|
</div>
|
||||||
|
<div id="variable-registry-table"></div>
|
||||||
|
<div id="variable-update-status" class="hidden">
|
||||||
|
<div class="setting">
|
||||||
|
<div class="setting-name">
|
||||||
|
Steps to update variables
|
||||||
|
</div>
|
||||||
|
<div class="setting-field">
|
||||||
|
<ol id="variable-update-steps"></ol>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
<div class="spacer"> </div>
|
||||||
|
<div class="settings-right">
|
||||||
|
<div class="setting">
|
||||||
|
<div class="setting-name">
|
||||||
|
Variables
|
||||||
|
</div>
|
||||||
|
<div class="setting-field">
|
||||||
|
<div id="affected-components-context"></div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
<div class="setting">
|
||||||
|
<div class="setting-name">
|
||||||
|
Referencing Processors
|
||||||
|
<div class="fa fa-question-circle" alt="Info" title="Processors referencing this variable."></div>
|
||||||
|
</div>
|
||||||
|
<div class="setting-field">
|
||||||
|
<ul id="variable-registry-affected-processors"></ul>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
<div class="setting">
|
||||||
|
<div class="setting-name">
|
||||||
|
Referencing Controller Services
|
||||||
|
<div class="fa fa-question-circle" alt="Info" title="Controller Services referencing this variable."></div>
|
||||||
|
</div>
|
||||||
|
<div class="setting-field">
|
||||||
|
<ul id="variable-registry-affected-controller-services"></ul>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
<div class="setting">
|
||||||
|
<div class="setting-name">
|
||||||
|
Unauthorized referencing components
|
||||||
|
<div class="fa fa-question-circle" alt="Info" title="Referencing components for which READ or WRITE permissions are not granted."></div>
|
||||||
|
</div>
|
||||||
|
<div class="setting-field">
|
||||||
|
<ul id="variable-registry-affected-unauthorized-components"></ul>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
<div id="new-variable-dialog" class="dialog cancellable small-dialog hidden">
|
||||||
|
<div class="dialog-content">
|
||||||
|
<div>
|
||||||
|
<div class="setting-name">Variable name</div>
|
||||||
|
<div class="setting-field new-variable-name-container">
|
||||||
|
<input id="new-variable-name" type="text"/>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
||||||
|
</div>
|
|
@ -79,6 +79,10 @@ ul.referencing-component-listing li {
|
||||||
white-space: nowrap;
|
white-space: nowrap;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
div.referencing-component-state {
|
||||||
|
width: 13px;
|
||||||
|
}
|
||||||
|
|
||||||
div.referencing-component-state.disabled:before {
|
div.referencing-component-state.disabled:before {
|
||||||
content: '\e802';
|
content: '\e802';
|
||||||
font-family: flowfont;
|
font-family: flowfont;
|
||||||
|
|
|
@ -212,6 +212,45 @@ div.progress-label {
|
||||||
height: 575px;
|
height: 575px;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/*
|
||||||
|
Variable Registry
|
||||||
|
*/
|
||||||
|
|
||||||
|
#variable-registry-dialog {
|
||||||
|
width: 850px;
|
||||||
|
height: 575px;
|
||||||
|
}
|
||||||
|
|
||||||
|
#variable-registry-dialog div.settings-left {
|
||||||
|
float: left;
|
||||||
|
width: 65%;
|
||||||
|
}
|
||||||
|
|
||||||
|
#variable-registry-dialog div.settings-right {
|
||||||
|
float: left;
|
||||||
|
width: 33%;
|
||||||
|
}
|
||||||
|
|
||||||
|
#variable-registry-table {
|
||||||
|
height: 400px;
|
||||||
|
}
|
||||||
|
|
||||||
|
#add-variable {
|
||||||
|
float: right;
|
||||||
|
margin-bottom: 4px;
|
||||||
|
font-size: 16px;
|
||||||
|
text-transform: uppercase;
|
||||||
|
}
|
||||||
|
|
||||||
|
li.affected-component-container {
|
||||||
|
margin-bottom: 3px;
|
||||||
|
height: 16px;
|
||||||
|
}
|
||||||
|
|
||||||
|
div.slick-cell div.overridden {
|
||||||
|
text-decoration: line-through;
|
||||||
|
}
|
||||||
|
|
||||||
/*
|
/*
|
||||||
General dialog styles.
|
General dialog styles.
|
||||||
*/
|
*/
|
||||||
|
@ -239,3 +278,15 @@ ul.result li {
|
||||||
float: left;
|
float: left;
|
||||||
width: 2%;
|
width: 2%;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
div.variable-step {
|
||||||
|
width: 16px;
|
||||||
|
height: 16px;
|
||||||
|
background-color: transparent;
|
||||||
|
float: right;
|
||||||
|
}
|
||||||
|
|
||||||
|
#variable-update-steps li {
|
||||||
|
width: 300px;
|
||||||
|
margin-bottom: 2px;
|
||||||
|
}
|
||||||
|
|
|
@ -1682,14 +1682,14 @@
|
||||||
// build the new property dialog
|
// build the new property dialog
|
||||||
var newPropertyDialogMarkup =
|
var newPropertyDialogMarkup =
|
||||||
'<div id="new-property-dialog" class="dialog cancellable small-dialog hidden">' +
|
'<div id="new-property-dialog" class="dialog cancellable small-dialog hidden">' +
|
||||||
'<div class="dialog-content">' +
|
'<div class="dialog-content">' +
|
||||||
'<div>' +
|
'<div>' +
|
||||||
'<div class="setting-name">Property name</div>' +
|
'<div class="setting-name">Property name</div>' +
|
||||||
'<div class="setting-field new-property-name-container">' +
|
'<div class="setting-field new-property-name-container">' +
|
||||||
'<input class="new-property-name" type="text"/>' +
|
'<input class="new-property-name" type="text"/>' +
|
||||||
'</div>' +
|
'</div>' +
|
||||||
'</div>' +
|
'</div>' +
|
||||||
'</div>' +
|
'</div>' +
|
||||||
'</div>';
|
'</div>';
|
||||||
|
|
||||||
var newPropertyDialog = $(newPropertyDialogMarkup).appendTo(options.dialogContainer);
|
var newPropertyDialog = $(newPropertyDialogMarkup).appendTo(options.dialogContainer);
|
||||||
|
|
|
@ -31,6 +31,7 @@
|
||||||
'nf.GoTo',
|
'nf.GoTo',
|
||||||
'nf.ng.Bridge',
|
'nf.ng.Bridge',
|
||||||
'nf.Shell',
|
'nf.Shell',
|
||||||
|
'nf.VariableRegistry',
|
||||||
'nf.ComponentState',
|
'nf.ComponentState',
|
||||||
'nf.Draggable',
|
'nf.Draggable',
|
||||||
'nf.Birdseye',
|
'nf.Birdseye',
|
||||||
|
@ -54,8 +55,8 @@
|
||||||
'nf.ComponentVersion',
|
'nf.ComponentVersion',
|
||||||
'nf.QueueListing',
|
'nf.QueueListing',
|
||||||
'nf.StatusHistory'],
|
'nf.StatusHistory'],
|
||||||
function ($, d3, nfCanvasUtils, nfCommon, nfDialog, nfClient, nfErrorHandler, nfClipboard, nfSnippet, nfGoto, nfNgBridge, nfShell, nfComponentState, nfDraggable, nfBirdseye, nfConnection, nfGraph, nfProcessGroupConfiguration, nfProcessorConfiguration, nfProcessorDetails, nfLabelConfiguration, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupDetails, nfPortConfiguration, nfPortDetails, nfConnectionConfiguration, nfConnectionDetails, nfPolicyManagement, nfRemoteProcessGroup, nfLabel, nfProcessor, nfRemoteProcessGroupPorts, nfComponentVersion, nfQueueListing, nfStatusHistory) {
|
function ($, d3, nfCanvasUtils, nfCommon, nfDialog, nfClient, nfErrorHandler, nfClipboard, nfSnippet, nfGoto, nfNgBridge, nfShell, nfVariableRegistry, nfComponentState, nfDraggable, nfBirdseye, nfConnection, nfGraph, nfProcessGroupConfiguration, nfProcessorConfiguration, nfProcessorDetails, nfLabelConfiguration, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupDetails, nfPortConfiguration, nfPortDetails, nfConnectionConfiguration, nfConnectionDetails, nfPolicyManagement, nfRemoteProcessGroup, nfLabel, nfProcessor, nfRemoteProcessGroupPorts, nfComponentVersion, nfQueueListing, nfStatusHistory) {
|
||||||
return (nf.Actions = factory($, d3, nfCanvasUtils, nfCommon, nfDialog, nfClient, nfErrorHandler, nfClipboard, nfSnippet, nfGoto, nfNgBridge, nfShell, nfComponentState, nfDraggable, nfBirdseye, nfConnection, nfGraph, nfProcessGroupConfiguration, nfProcessorConfiguration, nfProcessorDetails, nfLabelConfiguration, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupDetails, nfPortConfiguration, nfPortDetails, nfConnectionConfiguration, nfConnectionDetails, nfPolicyManagement, nfRemoteProcessGroup, nfLabel, nfProcessor, nfRemoteProcessGroupPorts, nfComponentVersion, nfQueueListing, nfStatusHistory));
|
return (nf.Actions = factory($, d3, nfCanvasUtils, nfCommon, nfDialog, nfClient, nfErrorHandler, nfClipboard, nfSnippet, nfGoto, nfNgBridge, nfShell, nfVariableRegistry, nfComponentState, nfDraggable, nfBirdseye, nfConnection, nfGraph, nfProcessGroupConfiguration, nfProcessorConfiguration, nfProcessorDetails, nfLabelConfiguration, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupDetails, nfPortConfiguration, nfPortDetails, nfConnectionConfiguration, nfConnectionDetails, nfPolicyManagement, nfRemoteProcessGroup, nfLabel, nfProcessor, nfRemoteProcessGroupPorts, nfComponentVersion, nfQueueListing, nfStatusHistory));
|
||||||
});
|
});
|
||||||
} else if (typeof exports === 'object' && typeof module === 'object') {
|
} else if (typeof exports === 'object' && typeof module === 'object') {
|
||||||
module.exports = (nf.Actions =
|
module.exports = (nf.Actions =
|
||||||
|
@ -71,6 +72,7 @@
|
||||||
require('nf.GoTo'),
|
require('nf.GoTo'),
|
||||||
require('nf.ng.Bridge'),
|
require('nf.ng.Bridge'),
|
||||||
require('nf.Shell'),
|
require('nf.Shell'),
|
||||||
|
require('nf.VariableRegistry'),
|
||||||
require('nf.ComponentState'),
|
require('nf.ComponentState'),
|
||||||
require('nf.Draggable'),
|
require('nf.Draggable'),
|
||||||
require('nf.Birdseye'),
|
require('nf.Birdseye'),
|
||||||
|
@ -107,6 +109,7 @@
|
||||||
root.nf.GoTo,
|
root.nf.GoTo,
|
||||||
root.nf.ng.Bridge,
|
root.nf.ng.Bridge,
|
||||||
root.nf.Shell,
|
root.nf.Shell,
|
||||||
|
root.nf.VariableRegistry,
|
||||||
root.nf.ComponentState,
|
root.nf.ComponentState,
|
||||||
root.nf.Draggable,
|
root.nf.Draggable,
|
||||||
root.nf.Birdseye,
|
root.nf.Birdseye,
|
||||||
|
@ -131,7 +134,7 @@
|
||||||
root.nf.QueueListing,
|
root.nf.QueueListing,
|
||||||
root.nf.StatusHistory);
|
root.nf.StatusHistory);
|
||||||
}
|
}
|
||||||
}(this, function ($, d3, nfCanvasUtils, nfCommon, nfDialog, nfClient, nfErrorHandler, nfClipboard, nfSnippet, nfGoto, nfNgBridge, nfShell, nfComponentState, nfDraggable, nfBirdseye, nfConnection, nfGraph, nfProcessGroupConfiguration, nfProcessorConfiguration, nfProcessorDetails, nfLabelConfiguration, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupDetails, nfPortConfiguration, nfPortDetails, nfConnectionConfiguration, nfConnectionDetails, nfPolicyManagement, nfRemoteProcessGroup, nfLabel, nfProcessor, nfRemoteProcessGroupPorts, nfComponentVersion, nfQueueListing, nfStatusHistory) {
|
}(this, function ($, d3, nfCanvasUtils, nfCommon, nfDialog, nfClient, nfErrorHandler, nfClipboard, nfSnippet, nfGoto, nfNgBridge, nfShell, nfVariableRegistry, nfComponentState, nfDraggable, nfBirdseye, nfConnection, nfGraph, nfProcessGroupConfiguration, nfProcessorConfiguration, nfProcessorDetails, nfLabelConfiguration, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupDetails, nfPortConfiguration, nfPortDetails, nfConnectionConfiguration, nfConnectionDetails, nfPolicyManagement, nfRemoteProcessGroup, nfLabel, nfProcessor, nfRemoteProcessGroupPorts, nfComponentVersion, nfQueueListing, nfStatusHistory) {
|
||||||
'use strict';
|
'use strict';
|
||||||
|
|
||||||
var config = {
|
var config = {
|
||||||
|
@ -1233,6 +1236,22 @@
|
||||||
nfComponentState.showState(processor, nfCanvasUtils.isConfigurable(selection));
|
nfComponentState.showState(processor, nfCanvasUtils.isConfigurable(selection));
|
||||||
},
|
},
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Opens the variable registry for the specified selection of the current group if the selection is emtpy.
|
||||||
|
*
|
||||||
|
* @param {selection} selection
|
||||||
|
*/
|
||||||
|
openVariableRegistry: function (selection) {
|
||||||
|
if (selection.empty()) {
|
||||||
|
nfVariableRegistry.showVariables(nfCanvasUtils.getGroupId());
|
||||||
|
} else if (selection.size() === 1) {
|
||||||
|
var selectionData = selection.datum();
|
||||||
|
if (nfCanvasUtils.isProcessGroup(selection)) {
|
||||||
|
nfVariableRegistry.showVariables(selectionData.id);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
},
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Views the state for the specified processor.
|
* Views the state for the specified processor.
|
||||||
*
|
*
|
||||||
|
|
|
@ -37,6 +37,7 @@
|
||||||
'nf.Snippet',
|
'nf.Snippet',
|
||||||
'nf.Actions',
|
'nf.Actions',
|
||||||
'nf.QueueListing',
|
'nf.QueueListing',
|
||||||
|
'nf.VariableRegistry',
|
||||||
'nf.ComponentState',
|
'nf.ComponentState',
|
||||||
'nf.ComponentVersion',
|
'nf.ComponentVersion',
|
||||||
'nf.Draggable',
|
'nf.Draggable',
|
||||||
|
@ -81,8 +82,8 @@
|
||||||
'nf.ng.Canvas.OperateCtrl',
|
'nf.ng.Canvas.OperateCtrl',
|
||||||
'nf.ng.BreadcrumbsDirective',
|
'nf.ng.BreadcrumbsDirective',
|
||||||
'nf.ng.DraggableDirective'],
|
'nf.ng.DraggableDirective'],
|
||||||
function ($, angular, nfCommon, nfCanvasUtils, nfErrorHandler, nfClient, nfClusterSummary, nfDialog, nfStorage, nfCanvas, nfGraph, nfContextMenu, nfQuickSelect, nfShell, nfSettings, nfActions, nfSnippet, nfQueueListing, nfComponentState, nfComponentVersion, nfDraggable, nfConnectable, nfStatusHistory, nfBirdseye, nfConnectionConfiguration, nfControllerService, nfReportingTask, nfPolicyManagement, nfProcessorConfiguration, nfProcessGroupConfiguration, nfControllerServices, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupPorts, nfPortConfiguration, nfLabelConfiguration, nfProcessorDetails, nfPortDetails, nfConnectionDetails, nfRemoteProcessGroupDetails, nfGoto, nfNgBridge, appCtrl, appConfig, serviceProvider, breadcrumbsCtrl, headerCtrl, flowStatusCtrl, globalMenuCtrl, toolboxCtrl, processorComponent, inputPortComponent, outputPortComponent, processGroupComponent, remoteProcessGroupComponent, funnelComponent, templateComponent, labelComponent, graphControlsCtrl, navigateCtrl, operateCtrl, breadcrumbsDirective, draggableDirective) {
|
function ($, angular, nfCommon, nfCanvasUtils, nfErrorHandler, nfClient, nfClusterSummary, nfDialog, nfStorage, nfCanvas, nfGraph, nfContextMenu, nfQuickSelect, nfShell, nfSettings, nfActions, nfSnippet, nfQueueListing, nfVariableRegistry, nfComponentState, nfComponentVersion, nfDraggable, nfConnectable, nfStatusHistory, nfBirdseye, nfConnectionConfiguration, nfControllerService, nfReportingTask, nfPolicyManagement, nfProcessorConfiguration, nfProcessGroupConfiguration, nfControllerServices, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupPorts, nfPortConfiguration, nfLabelConfiguration, nfProcessorDetails, nfPortDetails, nfConnectionDetails, nfRemoteProcessGroupDetails, nfGoto, nfNgBridge, appCtrl, appConfig, serviceProvider, breadcrumbsCtrl, headerCtrl, flowStatusCtrl, globalMenuCtrl, toolboxCtrl, processorComponent, inputPortComponent, outputPortComponent, processGroupComponent, remoteProcessGroupComponent, funnelComponent, templateComponent, labelComponent, graphControlsCtrl, navigateCtrl, operateCtrl, breadcrumbsDirective, draggableDirective) {
|
||||||
return factory($, angular, nfCommon, nfCanvasUtils, nfErrorHandler, nfClient, nfClusterSummary, nfDialog, nfStorage, nfCanvas, nfGraph, nfContextMenu, nfQuickSelect, nfShell, nfSettings, nfActions, nfSnippet, nfQueueListing, nfComponentState, nfComponentVersion, nfDraggable, nfConnectable, nfStatusHistory, nfBirdseye, nfConnectionConfiguration, nfControllerService, nfReportingTask, nfPolicyManagement, nfProcessorConfiguration, nfProcessGroupConfiguration, nfControllerServices, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupPorts, nfPortConfiguration, nfLabelConfiguration, nfProcessorDetails, nfPortDetails, nfConnectionDetails, nfRemoteProcessGroupDetails, nfGoto, nfNgBridge, appCtrl, appConfig, serviceProvider, breadcrumbsCtrl, headerCtrl, flowStatusCtrl, globalMenuCtrl, toolboxCtrl, processorComponent, inputPortComponent, outputPortComponent, processGroupComponent, remoteProcessGroupComponent, funnelComponent, templateComponent, labelComponent, graphControlsCtrl, navigateCtrl, operateCtrl, breadcrumbsDirective, draggableDirective);
|
return factory($, angular, nfCommon, nfCanvasUtils, nfErrorHandler, nfClient, nfClusterSummary, nfDialog, nfStorage, nfCanvas, nfGraph, nfContextMenu, nfQuickSelect, nfShell, nfSettings, nfActions, nfSnippet, nfQueueListing, nfVariableRegistry, nfComponentState, nfComponentVersion, nfDraggable, nfConnectable, nfStatusHistory, nfBirdseye, nfConnectionConfiguration, nfControllerService, nfReportingTask, nfPolicyManagement, nfProcessorConfiguration, nfProcessGroupConfiguration, nfControllerServices, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupPorts, nfPortConfiguration, nfLabelConfiguration, nfProcessorDetails, nfPortDetails, nfConnectionDetails, nfRemoteProcessGroupDetails, nfGoto, nfNgBridge, appCtrl, appConfig, serviceProvider, breadcrumbsCtrl, headerCtrl, flowStatusCtrl, globalMenuCtrl, toolboxCtrl, processorComponent, inputPortComponent, outputPortComponent, processGroupComponent, remoteProcessGroupComponent, funnelComponent, templateComponent, labelComponent, graphControlsCtrl, navigateCtrl, operateCtrl, breadcrumbsDirective, draggableDirective);
|
||||||
});
|
});
|
||||||
} else if (typeof exports === 'object' && typeof module === 'object') {
|
} else if (typeof exports === 'object' && typeof module === 'object') {
|
||||||
module.exports = factory(require('jquery'),
|
module.exports = factory(require('jquery'),
|
||||||
|
@ -103,6 +104,7 @@
|
||||||
require('nf.Actions'),
|
require('nf.Actions'),
|
||||||
require('nf.Snippet'),
|
require('nf.Snippet'),
|
||||||
require('nf.QueueListing'),
|
require('nf.QueueListing'),
|
||||||
|
require('nf.VariableRegistry'),
|
||||||
require('nf.ComponentState'),
|
require('nf.ComponentState'),
|
||||||
require('nf.ComponentVersion'),
|
require('nf.ComponentVersion'),
|
||||||
require('nf.Draggable'),
|
require('nf.Draggable'),
|
||||||
|
@ -166,6 +168,7 @@
|
||||||
root.nf.Actions,
|
root.nf.Actions,
|
||||||
root.nf.Snippet,
|
root.nf.Snippet,
|
||||||
root.nf.QueueListing,
|
root.nf.QueueListing,
|
||||||
|
root.nf.VariableRegistry,
|
||||||
root.nf.ComponentState,
|
root.nf.ComponentState,
|
||||||
root.nf.ComponentVersion,
|
root.nf.ComponentVersion,
|
||||||
root.nf.Draggable,
|
root.nf.Draggable,
|
||||||
|
@ -211,7 +214,7 @@
|
||||||
root.nf.ng.BreadcrumbsDirective,
|
root.nf.ng.BreadcrumbsDirective,
|
||||||
root.nf.ng.DraggableDirective);
|
root.nf.ng.DraggableDirective);
|
||||||
}
|
}
|
||||||
}(this, function ($, angular, nfCommon, nfCanvasUtils, nfErrorHandler, nfClient, nfClusterSummary, nfDialog, nfStorage, nfCanvas, nfGraph, nfContextMenu, nfQuickSelect, nfShell, nfSettings, nfActions, nfSnippet, nfQueueListing, nfComponentState, nfComponentVersion, nfDraggable, nfConnectable, nfStatusHistory, nfBirdseye, nfConnectionConfiguration, nfControllerService, nfReportingTask, nfPolicyManagement, nfProcessorConfiguration, nfProcessGroupConfiguration, nfControllerServices, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupPorts, nfPortConfiguration, nfLabelConfiguration, nfProcessorDetails, nfPortDetails, nfConnectionDetails, nfRemoteProcessGroupDetails, nfGoto, nfNgBridge, appCtrl, appConfig, serviceProvider, breadcrumbsCtrl, headerCtrl, flowStatusCtrl, globalMenuCtrl, toolboxCtrl, processorComponent, inputPortComponent, outputPortComponent, processGroupComponent, remoteProcessGroupComponent, funnelComponent, templateComponent, labelComponent, graphControlsCtrl, navigateCtrl, operateCtrl, breadcrumbsDirective, draggableDirective) {
|
}(this, function ($, angular, nfCommon, nfCanvasUtils, nfErrorHandler, nfClient, nfClusterSummary, nfDialog, nfStorage, nfCanvas, nfGraph, nfContextMenu, nfQuickSelect, nfShell, nfSettings, nfActions, nfSnippet, nfQueueListing, nfVariableRegistry, nfComponentState, nfComponentVersion, nfDraggable, nfConnectable, nfStatusHistory, nfBirdseye, nfConnectionConfiguration, nfControllerService, nfReportingTask, nfPolicyManagement, nfProcessorConfiguration, nfProcessGroupConfiguration, nfControllerServices, nfRemoteProcessGroupConfiguration, nfRemoteProcessGroupPorts, nfPortConfiguration, nfLabelConfiguration, nfProcessorDetails, nfPortDetails, nfConnectionDetails, nfRemoteProcessGroupDetails, nfGoto, nfNgBridge, appCtrl, appConfig, serviceProvider, breadcrumbsCtrl, headerCtrl, flowStatusCtrl, globalMenuCtrl, toolboxCtrl, processorComponent, inputPortComponent, outputPortComponent, processGroupComponent, remoteProcessGroupComponent, funnelComponent, templateComponent, labelComponent, graphControlsCtrl, navigateCtrl, operateCtrl, breadcrumbsDirective, draggableDirective) {
|
||||||
|
|
||||||
var config = {
|
var config = {
|
||||||
urls: {
|
urls: {
|
||||||
|
@ -347,6 +350,7 @@
|
||||||
nfSettings.init();
|
nfSettings.init();
|
||||||
nfActions.init();
|
nfActions.init();
|
||||||
nfQueueListing.init();
|
nfQueueListing.init();
|
||||||
|
nfVariableRegistry.init();
|
||||||
nfComponentState.init();
|
nfComponentState.init();
|
||||||
nfComponentVersion.init(nfSettings);
|
nfComponentVersion.init(nfSettings);
|
||||||
|
|
||||||
|
|
|
@ -238,6 +238,25 @@
|
||||||
}
|
}
|
||||||
},
|
},
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Queries for bulletins for the specified components.
|
||||||
|
*
|
||||||
|
* @param {array} componentIds
|
||||||
|
* @returns {deferred}
|
||||||
|
*/
|
||||||
|
queryBulletins: function (componentIds) {
|
||||||
|
var ids = componentIds.join('|');
|
||||||
|
|
||||||
|
return $.ajax({
|
||||||
|
type: 'GET',
|
||||||
|
url: '../nifi-api/flow/bulletin-board',
|
||||||
|
data: {
|
||||||
|
sourceId: ids
|
||||||
|
},
|
||||||
|
dataType: 'json'
|
||||||
|
}).fail(nfErrorHandler.handleAjaxError);
|
||||||
|
},
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Shows the specified component in the specified group.
|
* Shows the specified component in the specified group.
|
||||||
*
|
*
|
||||||
|
@ -282,6 +301,8 @@
|
||||||
});
|
});
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
|
|
||||||
|
return refreshGraph;
|
||||||
}
|
}
|
||||||
},
|
},
|
||||||
|
|
||||||
|
|
|
@ -64,6 +64,15 @@
|
||||||
return nfCanvasUtils.isConfigurable(selection);
|
return nfCanvasUtils.isConfigurable(selection);
|
||||||
};
|
};
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Determines whether the component in the specified selection has variables.
|
||||||
|
*
|
||||||
|
* @param {selection} selection The selection of currently selected components
|
||||||
|
*/
|
||||||
|
var hasVariables = function (selection) {
|
||||||
|
return selection.empty() || nfCanvasUtils.isProcessGroup(selection);
|
||||||
|
};
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Determines whether the component in the specified selection has configuration details.
|
* Determines whether the component in the specified selection has configuration details.
|
||||||
*
|
*
|
||||||
|
@ -537,6 +546,7 @@
|
||||||
{separator: true},
|
{separator: true},
|
||||||
{id: 'show-configuration-menu-item', condition: isConfigurable, menuItem: {clazz: 'fa fa-gear', text: 'Configure', action: 'showConfiguration'}},
|
{id: 'show-configuration-menu-item', condition: isConfigurable, menuItem: {clazz: 'fa fa-gear', text: 'Configure', action: 'showConfiguration'}},
|
||||||
{id: 'show-details-menu-item', condition: hasDetails, menuItem: {clazz: 'fa fa-gear', text: 'View configuration', action: 'showDetails'}},
|
{id: 'show-details-menu-item', condition: hasDetails, menuItem: {clazz: 'fa fa-gear', text: 'View configuration', action: 'showDetails'}},
|
||||||
|
{id: 'variable-registry-menu-item', condition: hasVariables, menuItem: {clazz: 'fa', text: 'Variables', action: 'openVariableRegistry'}},
|
||||||
{separator: true},
|
{separator: true},
|
||||||
{id: 'enter-group-menu-item', condition: isProcessGroup, menuItem: {clazz: 'fa fa-sign-in', text: 'Enter group', action: 'enterGroup'}},
|
{id: 'enter-group-menu-item', condition: isProcessGroup, menuItem: {clazz: 'fa fa-sign-in', text: 'Enter group', action: 'enterGroup'}},
|
||||||
{separator: true},
|
{separator: true},
|
||||||
|
|
|
@ -586,7 +586,7 @@
|
||||||
});
|
});
|
||||||
|
|
||||||
// query for the bulletins
|
// query for the bulletins
|
||||||
queryBulletins(referencingComponentIds).done(function (response) {
|
nfCanvasUtils.queryBulletins(referencingComponentIds).done(function (response) {
|
||||||
var bulletins = response.bulletinBoard.bulletins;
|
var bulletins = response.bulletinBoard.bulletins;
|
||||||
updateReferencingComponentBulletins(bulletins);
|
updateReferencingComponentBulletins(bulletins);
|
||||||
});
|
});
|
||||||
|
@ -622,25 +622,6 @@
|
||||||
createReferenceBlock('Unauthorized', unauthorized);
|
createReferenceBlock('Unauthorized', unauthorized);
|
||||||
};
|
};
|
||||||
|
|
||||||
/**
|
|
||||||
* Queries for bulletins for the specified components.
|
|
||||||
*
|
|
||||||
* @param {array} componentIds
|
|
||||||
* @returns {deferred}
|
|
||||||
*/
|
|
||||||
var queryBulletins = function (componentIds) {
|
|
||||||
var ids = componentIds.join('|');
|
|
||||||
|
|
||||||
return $.ajax({
|
|
||||||
type: 'GET',
|
|
||||||
url: '../nifi-api/flow/bulletin-board',
|
|
||||||
data: {
|
|
||||||
sourceId: ids
|
|
||||||
},
|
|
||||||
dataType: 'json'
|
|
||||||
}).fail(nfErrorHandler.handleAjaxError);
|
|
||||||
};
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Sets whether the specified controller service is enabled.
|
* Sets whether the specified controller service is enabled.
|
||||||
*
|
*
|
||||||
|
@ -688,7 +669,7 @@
|
||||||
return service.state === 'DISABLED';
|
return service.state === 'DISABLED';
|
||||||
}
|
}
|
||||||
}, function (service) {
|
}, function (service) {
|
||||||
return queryBulletins([service.id]);
|
return nfCanvasUtils.queryBulletins([service.id]);
|
||||||
}, pollCondition);
|
}, pollCondition);
|
||||||
|
|
||||||
// once the service has updated, resolve and render the updated service
|
// once the service has updated, resolve and render the updated service
|
||||||
|
@ -961,7 +942,7 @@
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
|
|
||||||
return queryBulletins(referencingSchedulableComponents);
|
return nfCanvasUtils.queryBulletins(referencingSchedulableComponents);
|
||||||
}, pollCondition);
|
}, pollCondition);
|
||||||
};
|
};
|
||||||
|
|
||||||
|
@ -1006,7 +987,7 @@
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
|
|
||||||
return queryBulletins(referencingSchedulableComponents);
|
return nfCanvasUtils.queryBulletins(referencingSchedulableComponents);
|
||||||
}, pollCondition);
|
}, pollCondition);
|
||||||
};
|
};
|
||||||
|
|
||||||
|
@ -1051,7 +1032,7 @@
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
|
|
||||||
return queryBulletins(referencingSchedulableComponents);
|
return nfCanvasUtils.queryBulletins(referencingSchedulableComponents);
|
||||||
}, pollCondition);
|
}, pollCondition);
|
||||||
};
|
};
|
||||||
|
|
||||||
|
@ -1164,7 +1145,7 @@
|
||||||
$('#disable-controller-service-dialog').modal('setButtonModel', buttons).modal('show');
|
$('#disable-controller-service-dialog').modal('setButtonModel', buttons).modal('show');
|
||||||
|
|
||||||
// load the bulletins
|
// load the bulletins
|
||||||
queryBulletins([controllerService.id]).done(function (response) {
|
nfCanvasUtils.queryBulletins([controllerService.id]).done(function (response) {
|
||||||
updateBulletins(response.bulletinBoard.bulletins, $('#disable-controller-service-bulletins'));
|
updateBulletins(response.bulletinBoard.bulletins, $('#disable-controller-service-bulletins'));
|
||||||
});
|
});
|
||||||
|
|
||||||
|
@ -1216,7 +1197,7 @@
|
||||||
$('#enable-controller-service-dialog').modal('setButtonModel', buttons).modal('show');
|
$('#enable-controller-service-dialog').modal('setButtonModel', buttons).modal('show');
|
||||||
|
|
||||||
// load the bulletins
|
// load the bulletins
|
||||||
queryBulletins([controllerService.id]).done(function (response) {
|
nfCanvasUtils.queryBulletins([controllerService.id]).done(function (response) {
|
||||||
updateBulletins(response.bulletinBoard.bulletins, $('#enable-controller-service-bulletins'));
|
updateBulletins(response.bulletinBoard.bulletins, $('#enable-controller-service-bulletins'));
|
||||||
});
|
});
|
||||||
|
|
||||||
|
|
File diff suppressed because it is too large
Load Diff
Loading…
Reference in New Issue