NIFI-3695 - created the nifi admin toolkit which includes shell scripts and classes to support notification and basic node management in standalone and clustered nifi.

This closes #1669.

Signed-off-by: Andy LoPresto <alopresto@apache.org>
This commit is contained in:
Yolanda M. Davis 2017-02-07 10:28:15 -05:00 committed by Andy LoPresto
parent ba2bdf8586
commit c0f0462e8b
No known key found for this signature in database
GPG Key ID: 6EC293152D90B61D
51 changed files with 3900 additions and 3 deletions

View File

@ -21,12 +21,14 @@ import org.apache.nifi.web.api.dto.BulletinDTO;
import org.apache.nifi.web.api.dto.ReadablePermission; import org.apache.nifi.web.api.dto.ReadablePermission;
import org.apache.nifi.web.api.dto.util.TimeAdapter; import org.apache.nifi.web.api.dto.util.TimeAdapter;
import javax.xml.bind.annotation.XmlRootElement;
import javax.xml.bind.annotation.adapters.XmlJavaTypeAdapter; import javax.xml.bind.annotation.adapters.XmlJavaTypeAdapter;
import java.util.Date; import java.util.Date;
/** /**
* A serialized representation of this class can be placed in the entity body of a request or response to or from the API. This particular entity holds a reference to a BulletinDTO. * A serialized representation of this class can be placed in the entity body of a request or response to or from the API. This particular entity holds a reference to a BulletinDTO.
*/ */
@XmlRootElement(name = "bulletinEntity")
public class BulletinEntity extends Entity implements ReadablePermission { public class BulletinEntity extends Entity implements ReadablePermission {
private Long id; private Long id;

View File

@ -361,6 +361,10 @@ public class FileAuthorizer extends AbstractPolicyBasedAuthorizer {
// grant access to the proxy resource // grant access to the proxy resource
addAccessPolicy(authorizations, ResourceType.Proxy.getValue(), jaxbNodeUser.getIdentifier(), WRITE_CODE); addAccessPolicy(authorizations, ResourceType.Proxy.getValue(), jaxbNodeUser.getIdentifier(), WRITE_CODE);
//grant access to controller resource
addAccessPolicy(authorizations, ResourceType.Controller.getValue(), jaxbNodeUser.getIdentifier(), READ_CODE);
addAccessPolicy(authorizations, ResourceType.Controller.getValue(), jaxbNodeUser.getIdentifier(), WRITE_CODE);
// grant the user read/write access data of the root group // grant the user read/write access data of the root group
if (rootGroupId != null) { if (rootGroupId != null) {
addAccessPolicy(authorizations, ResourceType.Data.getValue() + ResourceType.ProcessGroup.getValue() + "/" + rootGroupId, jaxbNodeUser.getIdentifier(), READ_CODE); addAccessPolicy(authorizations, ResourceType.Data.getValue() + ResourceType.ProcessGroup.getValue() + "/" + rootGroupId, jaxbNodeUser.getIdentifier(), READ_CODE);

View File

@ -24,6 +24,9 @@ import java.util.Collections;
import java.util.Comparator; import java.util.Comparator;
import java.util.List; import java.util.List;
import java.util.Map; import java.util.Map;
import java.util.stream.Collectors;
import com.google.common.collect.Lists;
public final class BulletinMerger { public final class BulletinMerger {
@ -71,7 +74,12 @@ public final class BulletinMerger {
} }
} }
Collections.sort(bulletinEntities, (BulletinEntity o1, BulletinEntity o2) -> { final List<BulletinEntity> entities = Lists.newArrayList();
final Map<String,List<BulletinEntity>> groupingEntities = bulletinEntities.stream().collect(Collectors.groupingBy(b -> b.getBulletin().getMessage()));
groupingEntities.values().stream().map(e -> e.get(0)).forEach(entities::add);
Collections.sort(entities, (BulletinEntity o1, BulletinEntity o2) -> {
final int timeComparison = o1.getTimestamp().compareTo(o2.getTimestamp()); final int timeComparison = o1.getTimestamp().compareTo(o2.getTimestamp());
if (timeComparison != 0) { if (timeComparison != 0) {
return timeComparison; return timeComparison;
@ -80,6 +88,6 @@ public final class BulletinMerger {
return o1.getNodeAddress().compareTo(o2.getNodeAddress()); return o1.getNodeAddress().compareTo(o2.getNodeAddress());
}); });
return bulletinEntities; return entities;
} }
} }

View File

@ -25,6 +25,7 @@ import org.apache.nifi.controller.service.ControllerServiceState;
import org.apache.nifi.groups.ProcessGroup; import org.apache.nifi.groups.ProcessGroup;
import org.apache.nifi.web.api.dto.AccessPolicyDTO; import org.apache.nifi.web.api.dto.AccessPolicyDTO;
import org.apache.nifi.web.api.dto.BulletinBoardDTO; import org.apache.nifi.web.api.dto.BulletinBoardDTO;
import org.apache.nifi.web.api.dto.BulletinDTO;
import org.apache.nifi.web.api.dto.BulletinQueryDTO; import org.apache.nifi.web.api.dto.BulletinQueryDTO;
import org.apache.nifi.web.api.dto.ClusterDTO; import org.apache.nifi.web.api.dto.ClusterDTO;
import org.apache.nifi.web.api.dto.ComponentHistoryDTO; import org.apache.nifi.web.api.dto.ComponentHistoryDTO;
@ -66,6 +67,7 @@ import org.apache.nifi.web.api.dto.search.SearchResultsDTO;
import org.apache.nifi.web.api.dto.status.ControllerStatusDTO; 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.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;
import org.apache.nifi.web.api.entity.ControllerBulletinsEntity; import org.apache.nifi.web.api.entity.ControllerBulletinsEntity;
@ -1049,6 +1051,15 @@ public interface NiFiServiceFacade {
*/ */
RemoteProcessGroupEntity deleteRemoteProcessGroup(Revision revision, String remoteProcessGroupId); RemoteProcessGroupEntity deleteRemoteProcessGroup(Revision revision, String remoteProcessGroupId);
/**
* Create a system bulletin
*
* @param bulletinDTO bulletin to send to users
* @param canRead allow users to read bulletin
*/
BulletinEntity createBulletin(final BulletinDTO bulletinDTO, final Boolean canRead);
// ---------------------------------------- // ----------------------------------------
// Funnel methods // Funnel methods
// ---------------------------------------- // ----------------------------------------

View File

@ -78,6 +78,7 @@ import org.apache.nifi.controller.service.ControllerServiceReference;
import org.apache.nifi.controller.service.ControllerServiceState; import org.apache.nifi.controller.service.ControllerServiceState;
import org.apache.nifi.controller.status.ProcessGroupStatus; import org.apache.nifi.controller.status.ProcessGroupStatus;
import org.apache.nifi.diagnostics.SystemDiagnostics; import org.apache.nifi.diagnostics.SystemDiagnostics;
import org.apache.nifi.events.BulletinFactory;
import org.apache.nifi.groups.ProcessGroup; import org.apache.nifi.groups.ProcessGroup;
import org.apache.nifi.groups.ProcessGroupCounts; import org.apache.nifi.groups.ProcessGroupCounts;
import org.apache.nifi.groups.RemoteProcessGroup; import org.apache.nifi.groups.RemoteProcessGroup;
@ -1380,6 +1381,12 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
}); });
} }
@Override
public BulletinEntity createBulletin(final BulletinDTO bulletinDTO, final Boolean canRead){
final Bulletin bulletin = BulletinFactory.createBulletin(bulletinDTO.getCategory(),bulletinDTO.getLevel(),bulletinDTO.getMessage());
bulletinRepository.addBulletin(bulletin);
return entityFactory.createBulletinEntity(dtoFactory.createBulletinDto(bulletin),canRead);
}
@Override @Override
public FunnelEntity createFunnel(final Revision revision, final String groupId, final FunnelDTO funnelDTO) { public FunnelEntity createFunnel(final Revision revision, final String groupId, final FunnelDTO funnelDTO) {

View File

@ -40,10 +40,12 @@ import org.apache.nifi.controller.FlowController;
import org.apache.nifi.web.IllegalClusterResourceRequestException; import org.apache.nifi.web.IllegalClusterResourceRequestException;
import org.apache.nifi.web.NiFiServiceFacade; import org.apache.nifi.web.NiFiServiceFacade;
import org.apache.nifi.web.Revision; import org.apache.nifi.web.Revision;
import org.apache.nifi.web.api.dto.BulletinDTO;
import org.apache.nifi.web.api.dto.ClusterDTO; import org.apache.nifi.web.api.dto.ClusterDTO;
import org.apache.nifi.web.api.dto.ControllerServiceDTO; import org.apache.nifi.web.api.dto.ControllerServiceDTO;
import org.apache.nifi.web.api.dto.NodeDTO; import org.apache.nifi.web.api.dto.NodeDTO;
import org.apache.nifi.web.api.dto.ReportingTaskDTO; import org.apache.nifi.web.api.dto.ReportingTaskDTO;
import org.apache.nifi.web.api.entity.BulletinEntity;
import org.apache.nifi.web.api.entity.ClusterEntity; import org.apache.nifi.web.api.entity.ClusterEntity;
import org.apache.nifi.web.api.entity.ControllerConfigurationEntity; import org.apache.nifi.web.api.entity.ControllerConfigurationEntity;
import org.apache.nifi.web.api.entity.ControllerServiceEntity; import org.apache.nifi.web.api.entity.ControllerServiceEntity;
@ -261,6 +263,7 @@ public class ControllerResource extends ApplicationResource {
@ApiResponse(code = 409, message = "The request was valid but NiFi was not in the appropriate state to process it. Retrying the same request later may be successful.") @ApiResponse(code = 409, message = "The request was valid but NiFi was not in the appropriate state to process it. Retrying the same request later may be successful.")
} }
) )
public Response createReportingTask( public Response createReportingTask(
@Context final HttpServletRequest httpServletRequest, @Context final HttpServletRequest httpServletRequest,
@ApiParam( @ApiParam(
@ -330,6 +333,71 @@ public class ControllerResource extends ApplicationResource {
); );
} }
/**
* Creates a Bulletin.
*
* @param httpServletRequest request
* @param requestBulletinEntity A bulletinEntity.
* @return A bulletinEntity.
*/
@POST
@Consumes(MediaType.APPLICATION_JSON)
@Produces(MediaType.APPLICATION_JSON)
@Path("bulletin")
@ApiOperation(
value = "Creates a new bulletin",
response = BulletinEntity.class,
authorizations = {
@Authorization(value = "Write - /controller", type = "")
}
)
@ApiResponses(
value = {
@ApiResponse(code = 400, message = "NiFi was unable to complete the request because it was invalid. The request should not be retried without modification."),
@ApiResponse(code = 401, message = "Client could not be authenticated."),
@ApiResponse(code = 403, message = "Client is not authorized to make this request."),
@ApiResponse(code = 409, message = "The request was valid but NiFi was not in the appropriate state to process it. Retrying the same request later may be successful.")
}
)
public Response createBulletin(
@Context final HttpServletRequest httpServletRequest,
@ApiParam(
value = "The reporting task configuration details.",
required = true
) final BulletinEntity requestBulletinEntity) {
if (requestBulletinEntity == null || requestBulletinEntity.getBulletin() == null) {
throw new IllegalArgumentException("Bulletin details must be specified.");
}
final BulletinDTO requestBulletin = requestBulletinEntity.getBulletin();
if (requestBulletin.getId() != null) {
throw new IllegalArgumentException("A bulletin ID cannot be specified.");
}
if (StringUtils.isBlank(requestBulletin.getMessage())) {
throw new IllegalArgumentException("The bulletin message must be specified.");
}
if (isReplicateRequest()) {
return replicate(HttpMethod.POST, requestBulletinEntity);
}
return withWriteLock(
serviceFacade,
requestBulletinEntity,
lookup -> {
authorizeController(RequestAction.WRITE);
},
null,
(bulletinEntity) -> {
final BulletinDTO bulletin = bulletinEntity.getBulletin();
final BulletinEntity entity = serviceFacade.createBulletin(bulletin,true);
return generateOkResponse(entity).build();
}
);
}
// ------------------- // -------------------
// controller services // controller services
// ------------------- // -------------------

View File

@ -23,7 +23,11 @@ import org.apache.nifi.authorization.user.NiFiUser
import org.apache.nifi.authorization.user.StandardNiFiUser import org.apache.nifi.authorization.user.StandardNiFiUser
import org.apache.nifi.authorization.user.NiFiUserDetails import org.apache.nifi.authorization.user.NiFiUserDetails
import org.apache.nifi.controller.service.ControllerServiceProvider import org.apache.nifi.controller.service.ControllerServiceProvider
import org.apache.nifi.reporting.Bulletin
import org.apache.nifi.reporting.BulletinRepository
import org.apache.nifi.reporting.ComponentType
import org.apache.nifi.web.api.dto.* import org.apache.nifi.web.api.dto.*
import org.apache.nifi.web.api.entity.BulletinEntity
import org.apache.nifi.web.api.entity.UserEntity import org.apache.nifi.web.api.entity.UserEntity
import org.apache.nifi.web.controller.ControllerFacade import org.apache.nifi.web.controller.ControllerFacade
import org.apache.nifi.web.dao.AccessPolicyDAO import org.apache.nifi.web.dao.AccessPolicyDAO
@ -36,7 +40,7 @@ import spock.lang.Ignore
import spock.lang.Specification import spock.lang.Specification
import spock.lang.Unroll import spock.lang.Unroll
@Ignore
class StandardNiFiServiceFacadeSpec extends Specification { class StandardNiFiServiceFacadeSpec extends Specification {
def setup() { def setup() {
@ -49,6 +53,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
SecurityContextHolder.getContext().setAuthentication(null); SecurityContextHolder.getContext().setAuthentication(null);
} }
@Ignore
@Unroll @Unroll
def "CreateUser: isAuthorized: #isAuthorized"() { def "CreateUser: isAuthorized: #isAuthorized"() {
given: given:
@ -87,6 +92,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
createUserDTO() | null | ResourceFactory.usersResource | false | AuthorizationResult.denied() createUserDTO() | null | ResourceFactory.usersResource | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "GetUser: isAuthorized: #isAuthorized"() { def "GetUser: isAuthorized: #isAuthorized"() {
given: given:
@ -134,6 +140,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
createUserDTO() | false | AuthorizationResult.denied() createUserDTO() | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "UpdateUser: isAuthorized: #isAuthorized, policy exists: #userExists"() { def "UpdateUser: isAuthorized: #isAuthorized, policy exists: #userExists"() {
given: given:
@ -188,6 +195,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
true | new Revision(1L, 'client1', 'root') | createUserDTO() | false | AuthorizationResult.denied() true | new Revision(1L, 'client1', 'root') | createUserDTO() | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "DeleteUser: isAuthorized: #isAuthorized, user exists: #userExists"() { def "DeleteUser: isAuthorized: #isAuthorized, user exists: #userExists"() {
given: given:
@ -239,6 +247,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
false | null | createUserDTO() | false | AuthorizationResult.denied() false | null | createUserDTO() | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "CreateUserGroup: isAuthorized: #isAuthorized"() { def "CreateUserGroup: isAuthorized: #isAuthorized"() {
given: given:
@ -307,6 +316,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
createUserGroupDTO() | false | [(ResourceFactory.userGroupsResource): AuthorizationResult.denied(), (ResourceFactory.usersResource): AuthorizationResult.denied()] createUserGroupDTO() | false | [(ResourceFactory.userGroupsResource): AuthorizationResult.denied(), (ResourceFactory.usersResource): AuthorizationResult.denied()]
} }
@Ignore
@Unroll @Unroll
def "GetUserGroup: isAuthorized: #isAuthorized"() { def "GetUserGroup: isAuthorized: #isAuthorized"() {
given: given:
@ -363,6 +373,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
new UserGroupDTO(id: '1', name: 'test group', users: [createUserEntity()]) | false | AuthorizationResult.denied() new UserGroupDTO(id: '1', name: 'test group', users: [createUserEntity()]) | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "UpdateUserGroup: isAuthorized: #isAuthorized, userGroupExists exists: #userGroupExists"() { def "UpdateUserGroup: isAuthorized: #isAuthorized, userGroupExists exists: #userGroupExists"() {
given: given:
@ -444,6 +455,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
[(ResourceFactory.userGroupsResource): AuthorizationResult.denied(), (ResourceFactory.usersResource): AuthorizationResult.denied()] [(ResourceFactory.userGroupsResource): AuthorizationResult.denied(), (ResourceFactory.usersResource): AuthorizationResult.denied()]
} }
@Ignore
@Unroll @Unroll
def "DeleteUserGroup: isAuthorized: #isAuthorized, userGroup exists: #userGroupExists"() { def "DeleteUserGroup: isAuthorized: #isAuthorized, userGroup exists: #userGroupExists"() {
given: given:
@ -521,6 +533,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
[(ResourceFactory.userGroupsResource): AuthorizationResult.denied(), (ResourceFactory.usersResource): AuthorizationResult.denied()] [(ResourceFactory.userGroupsResource): AuthorizationResult.denied(), (ResourceFactory.usersResource): AuthorizationResult.denied()]
} }
@Ignore
@Unroll @Unroll
def "CreateAccessPolicy: #isAuthorized"() { def "CreateAccessPolicy: #isAuthorized"() {
given: given:
@ -589,6 +602,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
new AccessPolicyDTO(id: '1', resource: ResourceFactory.flowResource.identifier, users: [createUserEntity()], canRead: true) | false | AuthorizationResult.denied() new AccessPolicyDTO(id: '1', resource: ResourceFactory.flowResource.identifier, users: [createUserEntity()], canRead: true) | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "GetAccessPolicy: isAuthorized: #isAuthorized"() { def "GetAccessPolicy: isAuthorized: #isAuthorized"() {
given: given:
@ -654,6 +668,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
new AccessPolicyDTO(id: '1', resource: ResourceFactory.flowResource.identifier, users: [createUserEntity()], canRead: true) | false | AuthorizationResult.denied() new AccessPolicyDTO(id: '1', resource: ResourceFactory.flowResource.identifier, users: [createUserEntity()], canRead: true) | false | AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "UpdateAccessPolicy: isAuthorized: #isAuthorized, policy exists: #hasPolicy"() { def "UpdateAccessPolicy: isAuthorized: #isAuthorized, policy exists: #hasPolicy"() {
given: given:
@ -741,6 +756,7 @@ class StandardNiFiServiceFacadeSpec extends Specification {
AuthorizationResult.denied() AuthorizationResult.denied()
} }
@Ignore
@Unroll @Unroll
def "DeleteAccessPolicy: isAuthorized: #isAuthorized, hasPolicy: #hasPolicy"() { def "DeleteAccessPolicy: isAuthorized: #isAuthorized, hasPolicy: #hasPolicy"() {
given: given:
@ -828,6 +844,44 @@ class StandardNiFiServiceFacadeSpec extends Specification {
AuthorizationResult.denied() AuthorizationResult.denied()
} }
def "CreateBulletin Successfully"() {
given:
def entityFactory = new EntityFactory()
def dtoFactory = new DtoFactory()
dtoFactory.setEntityFactory entityFactory
def authorizableLookup = Mock AuthorizableLookup
def controllerFacade = Mock ControllerFacade
def niFiServiceFacade = new StandardNiFiServiceFacade()
def bulletinRepository = Mock BulletinRepository
niFiServiceFacade.setAuthorizableLookup authorizableLookup
niFiServiceFacade.setDtoFactory dtoFactory
niFiServiceFacade.setEntityFactory entityFactory
niFiServiceFacade.setControllerFacade controllerFacade
niFiServiceFacade.setBulletinRepository bulletinRepository
def bulletinDto = new BulletinDTO()
bulletinDto.category = "SYSTEM"
bulletinDto.message = "test system message"
bulletinDto.level = "WARN"
def bulletinEntity
def retBulletinEntity = new BulletinEntity()
retBulletinEntity.bulletin = bulletinDto
when:
bulletinEntity = niFiServiceFacade.createBulletin(bulletinDto,true)
then:
1 * bulletinRepository.addBulletin(_ as Bulletin)
bulletinEntity
bulletinEntity.bulletin.message == bulletinDto.message
}
private UserGroupDTO createUserGroupDTO() { private UserGroupDTO createUserGroupDTO() {
new UserGroupDTO(id: 'group-1', name: 'test group', users: [createUserEntity()] as Set) new UserGroupDTO(id: 'group-1', name: 'test group', users: [createUserEntity()] as Set)
} }

View File

@ -0,0 +1,186 @@
<?xml version="1.0" encoding="UTF-8"?>
<!-- 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. -->
<project xmlns="http://maven.apache.org/POM/4.0.0"
xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
<parent>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-toolkit</artifactId>
<version>1.2.0-SNAPSHOT</version>
</parent>
<modelVersion>4.0.0</modelVersion>
<artifactId>nifi-toolkit-admin</artifactId>
<dependencies>
<dependency>
<groupId>commons-cli</groupId>
<artifactId>commons-cli</artifactId>
</dependency>
<dependency>
<groupId>com.google.guava</groupId>
<artifactId>guava</artifactId>
</dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-toolkit-tls</artifactId>
</dependency>
<dependency>
<groupId>com.sun.jersey</groupId>
<artifactId>jersey-client</artifactId>
</dependency>
<dependency>
<groupId>com.fasterxml.jackson.core</groupId>
<artifactId>jackson-databind</artifactId>
</dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-client-dto</artifactId>
<version>${client.version}</version>
</dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-properties</artifactId>
</dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-properties-loader</artifactId>
<exclusions>
<exclusion>
<groupId>ch.qos.logback</groupId>
<artifactId>logback-classic</artifactId>
</exclusion>
</exclusions>
</dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-security-utils</artifactId>
</dependency>
<dependency>
<groupId>org.codehaus.jackson</groupId>
<artifactId>jackson-mapper-asl</artifactId>
</dependency>
<dependency>
<groupId>com.fasterxml.jackson.jaxrs</groupId>
<artifactId>jackson-jaxrs-json-provider</artifactId>
<version>RELEASE</version>
</dependency>
<dependency>
<groupId>com.sun.jersey</groupId>
<artifactId>jersey-bundle</artifactId>
<version>RELEASE</version>
</dependency>
<dependency>
<groupId>com.sun.jersey</groupId>
<artifactId>jersey-json</artifactId>
<version>RELEASE</version>
</dependency>
<dependency>
<groupId>org.apache.commons</groupId>
<artifactId>commons-compress</artifactId>
</dependency>
<!-- Spock testing dependencies-->
<dependency>
<groupId>com.github.stefanbirkner</groupId>
<artifactId>system-rules</artifactId>
<version>1.16.0</version>
<scope>test</scope>
</dependency>
<dependency>
<groupId>org.spockframework</groupId>
<artifactId>spock-core</artifactId>
<scope>test</scope>
</dependency>
<dependency>
<groupId>cglib</groupId>
<artifactId>cglib-nodep</artifactId>
<scope>test</scope>
</dependency>
</dependencies>
<build>
<plugins>
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-compiler-plugin</artifactId>
<executions>
<execution>
<goals>
<goal>compile</goal>
<goal>testCompile</goal>
</goals>
<configuration>
<compilerId>groovy-eclipse-compiler</compilerId>
</configuration>
</execution>
</executions>
<configuration>
<source>1.8</source>
<target>1.8</target>
</configuration>
<dependencies>
<dependency>
<groupId>org.codehaus.groovy</groupId>
<artifactId>groovy-eclipse-compiler</artifactId>
<version>2.9.2-01</version>
</dependency>
<dependency>
<groupId>org.codehaus.groovy</groupId>
<artifactId>groovy-eclipse-batch</artifactId>
<version>2.4.3-01</version>
</dependency>
</dependencies>
</plugin>
<plugin>
<groupId>org.codehaus.mojo</groupId>
<artifactId>build-helper-maven-plugin</artifactId>
<version>1.5</version>
<executions>
<execution>
<id>add-source</id>
<phase>generate-sources</phase>
<goals>
<goal>add-source</goal>
</goals>
<configuration>
<sources>
<source>src/main/groovy</source>
</sources>
</configuration>
</execution>
<execution>
<id>add-test-source</id>
<phase>generate-test-sources</phase>
<goals>
<goal>add-test-source</goal>
</goals>
<configuration>
<sources>
<source>src/test/groovy</source>
</sources>
</configuration>
</execution>
</executions>
</plugin>
<plugin>
<groupId>org.apache.rat</groupId>
<artifactId>apache-rat-plugin</artifactId>
<configuration>
<excludes combine.children="append">
<exclude>src/test/resources/filemanager/myid</exclude>
</excludes>
</configuration>
</plugin>
</plugins>
</build>
</project>

View File

@ -0,0 +1,110 @@
/*
* 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.toolkit.admin
import org.apache.nifi.toolkit.admin.util.AdminUtil
import org.apache.commons.cli.HelpFormatter
import org.apache.commons.cli.Options
import org.apache.commons.lang3.SystemUtils
import org.apache.nifi.toolkit.admin.util.Version
import org.apache.nifi.util.StringUtils
import org.slf4j.Logger
import java.nio.file.Path
import java.nio.file.Paths
public abstract class AbstractAdminTool {
protected static final String JAVA_HOME = "JAVA_HOME"
protected static final String NIFI_TOOLKIT_HOME = "NIFI_TOOLKIT_HOME"
protected static final String SEP = System.lineSeparator()
protected Options options
protected String header
protected String footer
protected Boolean isVerbose
protected Logger logger
protected void setup(){
options = getOptions()
footer = buildFooter()
logger = getLogger()
}
protected String buildHeader(final String description ) {
"${SEP}${description}${SEP * 2}"
}
protected String buildFooter() {
"${SEP}Java home: ${System.getenv(JAVA_HOME)}${SEP}NiFi Toolkit home: ${System.getenv(NIFI_TOOLKIT_HOME)}"
}
public void printUsage(final String errorMessage) {
if (errorMessage) {
System.out.println(errorMessage)
System.out.println()
}
final HelpFormatter helpFormatter = new HelpFormatter()
helpFormatter.setWidth(160)
helpFormatter.printHelp(this.class.getCanonicalName(), this.header, options, footer, true)
}
protected abstract Options getOptions()
protected abstract Logger getLogger()
Properties getBootstrapConf(Path bootstrapConfFileName) {
Properties bootstrapProperties = new Properties()
File bootstrapConf = bootstrapConfFileName.toFile()
bootstrapProperties.load(new FileInputStream(bootstrapConf))
return bootstrapProperties
}
String getRelativeDirectory(String directory, String rootDirectory) {
if (directory.startsWith("./")) {
final String directoryUpdated = SystemUtils.IS_OS_WINDOWS ? File.separator + directory.substring(2,directory.length()) : directory.substring(1,directory.length())
rootDirectory + directoryUpdated
} else {
directory
}
}
Boolean supportedNiFiMinimumVersion(final String nifiConfDirName, final String nifiLibDirName, final String supportedMinimumVersion){
final File nifiConfDir = new File(nifiConfDirName)
final File nifiLibDir = new File (nifiLibDirName)
final String versionStr = AdminUtil.getNiFiVersion(nifiConfDir,nifiLibDir)
if(!StringUtils.isEmpty(versionStr)){
Version version = new Version(versionStr,".")
Version minVersion = new Version(supportedMinimumVersion,".")
Version.VERSION_COMPARATOR.compare(version,minVersion) >= 0
}else{
return false
}
}
Boolean supportedNiFiMinimumVersion(final String nifiCurrentDirName, final String supportedMinimumVersion){
final String bootstrapConfFileName = Paths.get(nifiCurrentDirName,"conf","bootstrap.conf").toString()
final File bootstrapConf = new File(bootstrapConfFileName)
final Properties bootstrapProperties = getBootstrapConf(Paths.get(bootstrapConfFileName))
final String parentPathName = bootstrapConf.getCanonicalFile().getParentFile().getParentFile().getCanonicalPath()
final String nifiConfDir = getRelativeDirectory(bootstrapProperties.getProperty("conf.dir"),parentPathName)
final String nifiLibDir = getRelativeDirectory(bootstrapProperties.getProperty("lib.dir"),parentPathName)
return supportedNiFiMinimumVersion(nifiConfDir,nifiLibDir,supportedMinimumVersion)
}
}

View File

@ -0,0 +1,27 @@
/*
* 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.toolkit.admin.client
import com.sun.jersey.api.client.Client
import org.apache.nifi.util.NiFiProperties
interface ClientFactory {
Client getClient(NiFiProperties niFiProperties, String nifiInstallDir) throws Exception
}

View File

@ -0,0 +1,172 @@
/*
* 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.toolkit.admin.client
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.config.ClientConfig
import com.sun.jersey.api.client.config.DefaultClientConfig
import com.sun.jersey.client.urlconnection.HTTPSProperties
import org.apache.commons.lang3.StringUtils
import org.apache.nifi.security.util.CertificateUtils
import org.apache.nifi.util.NiFiProperties
import org.slf4j.Logger
import org.slf4j.LoggerFactory
import javax.naming.ldap.LdapName
import javax.naming.ldap.Rdn
import javax.net.ssl.HostnameVerifier
import javax.net.ssl.KeyManagerFactory
import javax.net.ssl.SSLContext
import javax.net.ssl.SSLPeerUnverifiedException
import javax.net.ssl.SSLSession
import javax.net.ssl.TrustManagerFactory
import java.security.KeyManagementException
import java.security.KeyStore
import java.security.KeyStoreException
import java.security.NoSuchAlgorithmException
import java.security.SecureRandom
import java.security.UnrecoverableKeyException
import java.security.cert.Certificate
import java.security.cert.CertificateException
import java.security.cert.CertificateParsingException
import java.security.cert.X509Certificate
class NiFiClientFactory implements ClientFactory{
private static final Logger logger = LoggerFactory.getLogger(NiFiClientFactory.class)
static enum NiFiAuthType{ NONE, SSL }
public Client getClient(NiFiProperties niFiProperties, String nifiInstallDir) throws Exception {
final String authTypeStr = StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_HOST)) && StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT)) ? NiFiAuthType.NONE : NiFiAuthType.SSL;
final NiFiAuthType authType = NiFiAuthType.valueOf(authTypeStr);
SSLContext sslContext = null;
if (NiFiAuthType.SSL.equals(authType)) {
String keystore = niFiProperties.getProperty(NiFiProperties.SECURITY_KEYSTORE);
final String keystoreType = niFiProperties.getProperty(NiFiProperties.SECURITY_KEYSTORE_TYPE);
final String keystorePassword = niFiProperties.getProperty(NiFiProperties.SECURITY_KEYSTORE_PASSWD);
String truststore = niFiProperties.getProperty(NiFiProperties.SECURITY_TRUSTSTORE);
final String truststoreType = niFiProperties.getProperty(NiFiProperties.SECURITY_TRUSTSTORE_TYPE);
final String truststorePassword = niFiProperties.getProperty(NiFiProperties.SECURITY_TRUSTSTORE_PASSWD);
if(keystore.startsWith("./")){
keystore = keystore.replace("./",nifiInstallDir+"/")
}
if(truststore.startsWith("./")){
truststore = truststore.replace("./",nifiInstallDir+"/")
}
sslContext = createSslContext(
keystore.trim(),
keystorePassword.trim().toCharArray(),
keystoreType.trim(),
truststore.trim(),
truststorePassword.trim().toCharArray(),
truststoreType.trim(),
"TLS");
}
final ClientConfig config = new DefaultClientConfig();
if (sslContext != null) {
config.getProperties().put(HTTPSProperties.PROPERTY_HTTPS_PROPERTIES,new HTTPSProperties(new NiFiHostnameVerifier(), sslContext))
}
return Client.create(config)
}
static SSLContext createSslContext(
final String keystore, final char[] keystorePasswd, final String keystoreType,
final String truststore, final char[] truststorePasswd, final String truststoreType,
final String protocol)
throws KeyStoreException, IOException, NoSuchAlgorithmException, CertificateException,
UnrecoverableKeyException, KeyManagementException {
// prepare the keystore
final KeyStore keyStore = KeyStore.getInstance(keystoreType);
final InputStream keyStoreStream = new FileInputStream(keystore)
keyStore.load(keyStoreStream, keystorePasswd);
final KeyManagerFactory keyManagerFactory = KeyManagerFactory.getInstance(KeyManagerFactory.getDefaultAlgorithm());
keyManagerFactory.init(keyStore, keystorePasswd);
// prepare the truststore
final KeyStore trustStore = KeyStore.getInstance(truststoreType);
final InputStream trustStoreStream = new FileInputStream(truststore)
trustStore.load(trustStoreStream, truststorePasswd);
final TrustManagerFactory trustManagerFactory = TrustManagerFactory.getInstance(TrustManagerFactory.getDefaultAlgorithm());
trustManagerFactory.init(trustStore);
// initialize the ssl context
final SSLContext sslContext = SSLContext.getInstance(protocol);
sslContext.init(keyManagerFactory.getKeyManagers(), trustManagerFactory.getTrustManagers(), new SecureRandom());
return sslContext;
}
static class NiFiHostnameVerifier implements HostnameVerifier {
@Override
public boolean verify(final String hostname, final SSLSession ssls) {
if (ssls.getPeerCertificates() != null && ssls.getPeerCertificates().length > 0) {
try {
final Certificate peerCertificate = ssls.getPeerCertificates()[0]
final X509Certificate x509Cert = CertificateUtils.convertAbstractX509Certificate(peerCertificate)
final String dn = x509Cert.getSubjectDN().getName().trim()
final LdapName ln = new LdapName(dn)
final boolean match = ln.getRdns().any { Rdn rdn -> rdn.getType().equalsIgnoreCase("CN") && rdn.getValue().toString().equalsIgnoreCase(hostname)}
return match || getSubjectAlternativeNames(x509Cert).any { String san -> san.equalsIgnoreCase(hostname) }
} catch (final SSLPeerUnverifiedException | CertificateParsingException ex ) {
logger.warn("Hostname Verification encountered exception verifying hostname due to: " + ex, ex);
}
}else{
logger.warn("Peer certificates not found on ssl session ");
}
return false
}
private List<String> getSubjectAlternativeNames(final X509Certificate certificate) throws CertificateParsingException {
final Collection<List<?>> altNames = certificate.getSubjectAlternativeNames()
if (altNames == null) {
return new ArrayList<>()
}
final List<String> result = new ArrayList<>()
for (final List<?> generalName : altNames) {
final Object value = generalName.get(1)
if (value instanceof String) {
result.add(((String) value).toLowerCase())
}
}
return result
}
}
}

View File

@ -0,0 +1,144 @@
/*
* 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.toolkit.admin.client
import com.fasterxml.jackson.databind.ObjectMapper
import com.google.common.collect.Lists
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.ClientResponse
import com.sun.jersey.api.client.WebResource
import org.apache.nifi.util.NiFiProperties
import org.apache.nifi.util.StringUtils
import org.apache.nifi.web.api.dto.NodeDTO
import org.apache.nifi.web.api.dto.util.DateTimeAdapter
import org.apache.nifi.web.api.entity.ClusterEntity
import org.apache.nifi.web.api.entity.NodeEntity
import org.slf4j.Logger
import org.slf4j.LoggerFactory
import java.text.SimpleDateFormat
public class NiFiClientUtil {
private static final Logger logger = LoggerFactory.getLogger(NiFiClientUtil.class)
private final static String GET_CLUSTER_ENDPOINT ="/nifi-api/controller/cluster"
public static Boolean isCluster(final NiFiProperties niFiProperties){
String clusterNode = niFiProperties.getProperty(NiFiProperties.CLUSTER_IS_NODE)
return Boolean.valueOf(clusterNode)
}
public static String getUrl(NiFiProperties niFiProperties, String endpoint){
final StringBuilder urlBuilder = new StringBuilder();
if(!StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT))){
urlBuilder.append("https://")
urlBuilder.append(StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_HOST)) ? "localhost": niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_HOST))
urlBuilder.append(":")
urlBuilder.append(StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT)) ? "8081" : niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT))
}else{
urlBuilder.append("http://")
urlBuilder.append(StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTP_HOST)) ? "localhost": niFiProperties.getProperty(NiFiProperties.WEB_HTTP_HOST))
urlBuilder.append(":")
urlBuilder.append(StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT)) ? "8080": niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT))
}
if(!StringUtils.isEmpty(endpoint)) {
urlBuilder.append(endpoint)
}
urlBuilder.toString()
}
public static String getUrl(NiFiProperties niFiProperties, NodeDTO nodeDTO, String endpoint){
final StringBuilder urlBuilder = new StringBuilder();
if(!StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.WEB_HTTPS_PORT))){
urlBuilder.append("https://")
}else{
urlBuilder.append("http://")
}
urlBuilder.append(nodeDTO.address)
urlBuilder.append(":")
urlBuilder.append(nodeDTO.apiPort)
if(!StringUtils.isEmpty(endpoint)) {
urlBuilder.append(endpoint)
}
urlBuilder.toString()
}
public static ClusterEntity getCluster(final Client client, NiFiProperties niFiProperties, List<String> activeUrls){
if(activeUrls.isEmpty()){
final String url = getUrl(niFiProperties,null)
activeUrls.add(url)
}
for(String activeUrl: activeUrls) {
try {
String url = activeUrl + GET_CLUSTER_ENDPOINT
final WebResource webResource = client.resource(url)
final ClientResponse response = webResource.type("application/json").get(ClientResponse.class)
Integer status = response.getStatus()
if (status != 200) {
if (status == 404) {
logger.warn("This node is not attached to a cluster. Please connect to a node that is attached to the cluster for information")
} else {
logger.warn("Failed with HTTP error code: {}, message: {}", status, response.getStatusInfo().getReasonPhrase())
}
} else if (status == 200) {
return response.getEntity(ClusterEntity.class)
}
}catch(Exception ex){
logger.warn("Exception occurred during connection attempt: {}",ex.localizedMessage)
}
}
throw new RuntimeException("Unable to obtain cluster information")
}
public static List<String> getActiveClusterUrls(final Client client, NiFiProperties niFiProperties){
final ClusterEntity clusterEntity = getCluster(client, niFiProperties, Lists.newArrayList())
final List<NodeDTO> activeNodes = clusterEntity.cluster.nodes.findAll{ it.status == "CONNECTED" }
final List<String> activeUrls = Lists.newArrayList()
activeNodes.each {
activeUrls.add(getUrl(niFiProperties,it, null))
}
activeUrls
}
public static String convertToJson(NodeDTO nodeDTO){
ObjectMapper om = new ObjectMapper()
om.setDateFormat(new SimpleDateFormat(DateTimeAdapter.DEFAULT_DATE_TIME_FORMAT));
NodeEntity ne = new NodeEntity()
ne.setNode(nodeDTO)
return om.writeValueAsString(ne)
}
}

View File

@ -0,0 +1,289 @@
/*
* 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.toolkit.admin.nodemanager
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.ClientResponse
import com.sun.jersey.api.client.WebResource
import org.apache.nifi.toolkit.admin.AbstractAdminTool
import org.apache.nifi.toolkit.admin.client.NiFiClientUtil
import org.apache.commons.cli.CommandLine
import org.apache.commons.cli.DefaultParser
import org.apache.commons.cli.Option
import org.apache.commons.cli.Options
import org.apache.commons.cli.ParseException
import org.apache.nifi.properties.NiFiPropertiesLoader
import org.apache.nifi.toolkit.admin.client.ClientFactory
import org.apache.nifi.toolkit.admin.client.NiFiClientFactory
import org.apache.nifi.util.NiFiProperties
import org.apache.nifi.util.StringUtils
import org.apache.nifi.web.api.dto.NodeDTO
import org.apache.nifi.web.api.entity.ClusterEntity
import org.apache.nifi.web.api.entity.NodeEntity
import org.slf4j.Logger
import org.slf4j.LoggerFactory
import java.nio.file.Paths
public class NodeManagerTool extends AbstractAdminTool {
private static final String DEFAULT_DESCRIPTION = "This tool is used to manage nodes within a cluster. Supported functionality will remove node from cluster. "
private static final String HELP_ARG = "help"
private static final String VERBOSE_ARG = "verbose"
private static final String BOOTSTRAP_CONF = "bootstrapConf"
private static final String NIFI_INSTALL_DIR = "nifiInstallDir"
private static final String CLUSTER_URLS = "clusterUrls"
private static final String REMOVE = "remove"
private static final String DISCONNECT = "disconnect"
private static final String CONNECT = "connect"
private static final String OPERATION = "operation"
private final static String NODE_ENDPOINT = "/nifi-api/controller/cluster/nodes"
private final static String SUPPORTED_MINIMUM_VERSION = "1.0.0"
static enum STATUS {DISCONNECTING,CONNECTING,CONNECTED}
NodeManagerTool() {
header = buildHeader(DEFAULT_DESCRIPTION)
setup()
}
NodeManagerTool(final String description){
this.header = buildHeader(description)
setup()
}
@Override
protected Logger getLogger() {
LoggerFactory.getLogger(NodeManagerTool.class)
}
protected Options getOptions(){
final Options options = new Options()
options.addOption(Option.builder("h").longOpt(HELP_ARG).desc("Print help info").build())
options.addOption(Option.builder("v").longOpt(VERBOSE_ARG).desc("Set mode to verbose (default is false)").build())
options.addOption(Option.builder("b").longOpt(BOOTSTRAP_CONF).hasArg().desc("Existing Bootstrap Configuration file").build())
options.addOption(Option.builder("d").longOpt(NIFI_INSTALL_DIR).hasArg().desc("NiFi Installation Directory").build())
options.addOption(Option.builder("o").longOpt(OPERATION).hasArg().desc("Operation to connect, disconnect or remove node from cluster").build())
options.addOption(Option.builder("u").longOpt(CLUSTER_URLS).hasArg().desc("List of active urls for the cluster").build())
options
}
NodeDTO getCurrentNode(ClusterEntity clusterEntity, NiFiProperties niFiProperties){
final List<NodeDTO> nodeDTOs = clusterEntity.cluster.nodes
final String nodeHost = StringUtils.isEmpty(niFiProperties.getProperty(NiFiProperties.CLUSTER_NODE_ADDRESS)) ?
"localhost":niFiProperties.getProperty(NiFiProperties.CLUSTER_NODE_ADDRESS)
return nodeDTOs.find{ it.address == nodeHost }
}
NodeEntity updateNode(final String url, final Client client, final NodeDTO nodeDTO, final STATUS nodeStatus){
final WebResource webResource = client.resource(url)
nodeDTO.status = nodeStatus
String json = NiFiClientUtil.convertToJson(nodeDTO)
if(isVerbose){
logger.info("Sending node info for update: " + json)
}
final ClientResponse response = webResource.type("application/json").put(ClientResponse.class,json)
if(response.getStatus() != 200){
throw new RuntimeException("Failed with HTTP error code: " + response.getStatus())
}else{
response.getEntity(NodeEntity.class)
}
}
void deleteNode(final String url, final Client client){
final WebResource webResource = client.resource(url)
if(isVerbose){
logger.info("Attempting to delete node" )
}
final ClientResponse response = webResource.type("application/json").delete(ClientResponse.class)
if(response.getStatus() != 200){
throw new RuntimeException("Failed with HTTP error code: " + response.getStatus())
}
}
void disconnectNode(final Client client, NiFiProperties niFiProperties, List<String> activeUrls){
final ClusterEntity clusterEntity = NiFiClientUtil.getCluster(client, niFiProperties, activeUrls)
NodeDTO currentNode = getCurrentNode(clusterEntity,niFiProperties)
for(String activeUrl: activeUrls) {
try {
final String url = activeUrl + NODE_ENDPOINT + File.separator + currentNode.nodeId
updateNode(url, client, currentNode, STATUS.DISCONNECTING)
return
} catch (Exception ex){
logger.warn("Could not connect to node on "+activeUrl+". Exception: "+ex.toString())
}
}
throw new RuntimeException("Could not successfully complete request")
}
void connectNode(final Client client, NiFiProperties niFiProperties,List<String> activeUrls){
final ClusterEntity clusterEntity = NiFiClientUtil.getCluster(client, niFiProperties, activeUrls)
NodeDTO currentNode = getCurrentNode(clusterEntity,niFiProperties)
for(String activeUrl: activeUrls) {
try {
final String url = activeUrl + NODE_ENDPOINT + File.separator + currentNode.nodeId
updateNode(url, client, currentNode, STATUS.CONNECTING)
return
} catch (Exception ex){
logger.warn("Could not connect to node on "+activeUrl+". Exception: "+ex.toString())
}
}
throw new RuntimeException("Could not successfully complete request")
}
void removeNode(final Client client, NiFiProperties niFiProperties, List<String> activeUrls){
final ClusterEntity clusterEntity = NiFiClientUtil.getCluster(client, niFiProperties, activeUrls)
NodeDTO currentNode = getCurrentNode(clusterEntity,niFiProperties)
if(currentNode != null) {
for (String activeUrl : activeUrls) {
try {
final String url = activeUrl + NODE_ENDPOINT + File.separator + currentNode.nodeId
if(isVerbose){
logger.info("Attempting to connect to cluster with url:" + url)
}
if(currentNode.status == "CONNECTED") {
currentNode = updateNode(url, client, currentNode, STATUS.DISCONNECTING).node
}
if(currentNode.status == "DISCONNECTED") {
deleteNode(url, client)
}
if(isVerbose){
logger.info("Node removed from cluster successfully.")
}
return
}catch (Exception ex){
logger.warn("Could not connect to node on "+activeUrl+". Exception: "+ex.toString())
}
}
throw new RuntimeException("Could not successfully complete request")
}else{
throw new RuntimeException("Current node could not be found in the cluster")
}
}
void parse(final ClientFactory clientFactory, final String[] args) throws ParseException, UnsupportedOperationException, IllegalArgumentException {
final CommandLine commandLine = new DefaultParser().parse(options,args)
if (commandLine.hasOption(HELP_ARG)){
printUsage(null)
}else{
if(commandLine.hasOption(BOOTSTRAP_CONF) && commandLine.hasOption(NIFI_INSTALL_DIR) && commandLine.hasOption(OPERATION)) {
if(commandLine.hasOption(VERBOSE_ARG)){
this.isVerbose = true;
}
final String bootstrapConfFileName = commandLine.getOptionValue(BOOTSTRAP_CONF)
final File bootstrapConf = new File(bootstrapConfFileName)
Properties bootstrapProperties = getBootstrapConf(Paths.get(bootstrapConfFileName))
String nifiConfDir = getRelativeDirectory(bootstrapProperties.getProperty("conf.dir"), bootstrapConf.getCanonicalFile().getParentFile().getParentFile().getCanonicalPath())
String nifiLibDir = getRelativeDirectory(bootstrapProperties.getProperty("lib.dir"), bootstrapConf.getCanonicalFile().getParentFile().getParentFile().getCanonicalPath())
String nifiPropertiesFileName = nifiConfDir + File.separator +"nifi.properties"
final String key = NiFiPropertiesLoader.extractKeyFromBootstrapFile(bootstrapConfFileName)
final NiFiProperties niFiProperties = NiFiPropertiesLoader.withKey(key).load(nifiPropertiesFileName)
final String nifiInstallDir = commandLine.getOptionValue(NIFI_INSTALL_DIR)
if(supportedNiFiMinimumVersion(nifiConfDir,nifiLibDir,SUPPORTED_MINIMUM_VERSION) && NiFiClientUtil.isCluster(niFiProperties)){
final Client client = clientFactory.getClient(niFiProperties,nifiInstallDir)
final String operation = commandLine.getOptionValue(OPERATION)
if(isVerbose){
logger.info("Starting {} request",operation)
}
List<String> activeUrls
if(commandLine.hasOption(CLUSTER_URLS)){
final String urlList = commandLine.getOptionValue(CLUSTER_URLS)
activeUrls = urlList.tokenize(',')
}else{
activeUrls = NiFiClientUtil.getActiveClusterUrls(client,niFiProperties)
}
if(isVerbose){
logger.info("Using active urls {} for communication.",activeUrls)
}
if(operation.toLowerCase().equals(REMOVE)){
removeNode(client,niFiProperties,activeUrls)
}
else if(operation.toLowerCase().equals(DISCONNECT)){
disconnectNode(client,niFiProperties,activeUrls)
}
else if(operation.toLowerCase().equals(CONNECT)){
connectNode(client,niFiProperties,activeUrls)
}
else{
throw new ParseException("Invalid operation provided: " + operation)
}
}else{
throw new UnsupportedOperationException("Node Manager Tool only supports clustered instance of NiFi running versions 1.0.0 or higher.")
}
}else if(!commandLine.hasOption(BOOTSTRAP_CONF)){
throw new ParseException("Missing -b option")
}else if(!commandLine.hasOption(NIFI_INSTALL_DIR)){
throw new ParseException("Missing -d option")
}else{
throw new ParseException("Missing -o option")
}
}
}
public static void main(String[] args) {
final NodeManagerTool tool = new NodeManagerTool()
final ClientFactory clientFactory = new NiFiClientFactory()
try{
tool.parse(clientFactory,args)
} catch (ParseException | RuntimeException e ) {
tool.printUsage(e.getLocalizedMessage());
System.exit(1)
}
System.exit(0)
}
}

View File

@ -0,0 +1,181 @@
/*
* 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.toolkit.admin.notify
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.ClientResponse
import com.sun.jersey.api.client.WebResource
import org.apache.commons.lang3.StringUtils
import org.apache.nifi.toolkit.admin.client.NiFiClientUtil
import org.apache.commons.cli.CommandLine
import org.apache.commons.cli.DefaultParser
import org.apache.commons.cli.Option
import org.apache.commons.cli.Options
import org.apache.commons.cli.ParseException
import org.apache.nifi.properties.NiFiPropertiesLoader
import org.apache.nifi.toolkit.admin.AbstractAdminTool
import org.apache.nifi.toolkit.admin.client.ClientFactory
import org.apache.nifi.toolkit.admin.client.NiFiClientFactory
import org.apache.nifi.util.NiFiProperties
import org.apache.nifi.web.api.dto.BulletinDTO
import org.apache.nifi.web.api.entity.BulletinEntity
import org.slf4j.Logger
import org.slf4j.LoggerFactory
import java.nio.file.Paths
public class NotificationTool extends AbstractAdminTool {
private static final String DEFAULT_DESCRIPTION = "This tool is used to send notifications (bulletins) to a NiFi cluster. "
private static final String HELP_ARG = "help"
private static final String VERBOSE_ARG = "verbose"
private static final String BOOTSTRAP_CONF = "bootstrapConf"
private static final String NIFI_INSTALL_DIR = "nifiInstallDir"
private static final String NOTIFICATION_MESSAGE = "message"
private static final String NOTIFICATION_LEVEL = "level"
private final static String NOTIFICATION_ENDPOINT ="/nifi-api/controller/bulletin"
private final static String SUPPORTED_MINIMUM_VERSION = "1.2.0"
NotificationTool() {
header = buildHeader(DEFAULT_DESCRIPTION)
setup()
}
NotificationTool(final String description){
header = buildHeader(description)
setup()
}
@Override
protected Logger getLogger() {
LoggerFactory.getLogger(NotificationTool.class)
}
protected Options getOptions(){
final Options options = new Options()
options.addOption(Option.builder("h").longOpt(HELP_ARG).desc("Print help info").build())
options.addOption(Option.builder("v").longOpt(VERBOSE_ARG).desc("Set mode to verbose (default is false)").build())
options.addOption(Option.builder("b").longOpt(BOOTSTRAP_CONF).hasArg().desc("Existing Bootstrap Configuration file").build())
options.addOption(Option.builder("d").longOpt(NIFI_INSTALL_DIR).hasArg().desc("NiFi Installation Directory").build())
options.addOption(Option.builder("m").longOpt(NOTIFICATION_MESSAGE).hasArg().desc("Notification message for nifi instance or cluster").build())
options.addOption(Option.builder("l").longOpt(NOTIFICATION_LEVEL).required(false).hasArg().desc("Level for notification bulletin INFO,WARN,ERROR").build())
options
}
void notifyCluster(final ClientFactory clientFactory, final String nifiPropertiesFile, final String bootstrapConfFile, final String nifiInstallDir, final String message, final String level){
if(isVerbose){
logger.info("Loading nifi properties for host information")
}
final String key = NiFiPropertiesLoader.extractKeyFromBootstrapFile(bootstrapConfFile)
final NiFiProperties niFiProperties = NiFiPropertiesLoader.withKey(key).load(nifiPropertiesFile)
final Client client = clientFactory.getClient(niFiProperties,nifiInstallDir)
final String url = NiFiClientUtil.getUrl(niFiProperties,NOTIFICATION_ENDPOINT)
final WebResource webResource = client.resource(url)
if(isVerbose){
logger.info("Contacting node at url:" + url)
}
final BulletinEntity bulletinEntity = new BulletinEntity()
final BulletinDTO bulletinDTO = new BulletinDTO()
bulletinDTO.message = message
bulletinDTO.category = "NOTICE"
bulletinDTO.level = StringUtils.isEmpty(level) ? "INFO" : level
bulletinEntity.bulletin = bulletinDTO
final ClientResponse response = webResource.type("application/json").post(ClientResponse.class, bulletinEntity)
Integer status = response.getStatus()
if(status != 200){
if(status == 404){
throw new RuntimeException("The notification feature is not supported by each node in the cluster")
}else{
throw new RuntimeException("Failed with HTTP error code: " + status)
}
}
}
void parse(final ClientFactory clientFactory, final String[] args) throws ParseException, UnsupportedOperationException {
final CommandLine commandLine = new DefaultParser().parse(options,args)
if (commandLine.hasOption(HELP_ARG)){
printUsage(null)
}else{
if(commandLine.hasOption(BOOTSTRAP_CONF) && commandLine.hasOption(NOTIFICATION_MESSAGE) && commandLine.hasOption(NIFI_INSTALL_DIR)) {
if(commandLine.hasOption(VERBOSE_ARG)){
this.isVerbose = true;
}
final String bootstrapConfFileName = commandLine.getOptionValue(BOOTSTRAP_CONF)
final File bootstrapConf = new File(bootstrapConfFileName)
final Properties bootstrapProperties = getBootstrapConf(Paths.get(bootstrapConfFileName))
final String parentPathName = bootstrapConf.getCanonicalFile().getParentFile().getParentFile().getCanonicalPath()
final String nifiConfDir = getRelativeDirectory(bootstrapProperties.getProperty("conf.dir"),parentPathName)
final String nifiLibDir = getRelativeDirectory(bootstrapProperties.getProperty("lib.dir"),parentPathName)
final String nifiPropertiesFileName = nifiConfDir + File.separator +"nifi.properties"
final String notificationMessage = commandLine.getOptionValue(NOTIFICATION_MESSAGE)
final String notificationLevel = commandLine.getOptionValue(NOTIFICATION_LEVEL)
final String nifiInstallDir = commandLine.getOptionValue(NIFI_INSTALL_DIR)
if(supportedNiFiMinimumVersion(nifiConfDir, nifiLibDir, SUPPORTED_MINIMUM_VERSION)){
if(isVerbose){
logger.info("Attempting to connect with nifi using properties:", nifiPropertiesFileName)
}
notifyCluster(clientFactory, nifiPropertiesFileName, bootstrapConfFileName,nifiInstallDir,notificationMessage,notificationLevel)
if(isVerbose) {
logger.info("Message sent successfully to NiFi.")
}
}else{
throw new UnsupportedOperationException("Notification Tool only supports NiFi versions 1.2.0 and above")
}
}else if(!commandLine.hasOption(BOOTSTRAP_CONF)){
throw new ParseException("Missing -b option")
}else if(!commandLine.hasOption(NIFI_INSTALL_DIR)){
throw new ParseException("Missing -d option")
}else{
throw new ParseException("Missing -m option")
}
}
}
public static void main(String[] args) {
final NotificationTool tool = new NotificationTool()
final ClientFactory clientFactory = new NiFiClientFactory()
try{
tool.parse(clientFactory,args)
} catch (ParseException | UnsupportedOperationException e) {
tool.printUsage(e.message);
System.exit(1)
}
System.exit(0)
}
}

View File

@ -0,0 +1,69 @@
/*
* 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.toolkit.admin.util
import org.apache.commons.compress.archivers.zip.ZipArchiveEntry
import org.apache.commons.compress.archivers.zip.ZipFile
import org.apache.commons.lang3.StringUtils
class AdminUtil {
protected static String getNiFiVersionFromNar(final File nifiLibDir){
if(nifiLibDir.isDirectory()){
File[] files = nifiLibDir.listFiles(new FilenameFilter() {
@Override
boolean accept(File dir, String name) {
name.startsWith("nifi-framework-nar")
}
})
if(files.length == 1){
final ZipFile zipFile = new ZipFile(files[0])
final ZipArchiveEntry archiveEntry = zipFile.getEntry("META-INF/MANIFEST.MF")
final InputStream is = zipFile.getInputStream(archiveEntry)
final Properties manifestProperties = new Properties()
manifestProperties.load(is)
String version = manifestProperties.get("Nar-Version")
zipFile.close()
return StringUtils.isEmpty(version)? null : version
}
}
null
}
protected static String getNiFiVersionFromProperties(final File nifiConfDir) {
final String nifiPropertiesFileName = nifiConfDir.getAbsolutePath() + File.separator +"nifi.properties"
final File nifiPropertiesFile = new File(nifiPropertiesFileName)
final Properties nifiProperties = new Properties()
nifiProperties.load(new FileInputStream(nifiPropertiesFile))
nifiProperties.getProperty("nifi.version")
}
public static String getNiFiVersion(final File nifiConfDir, final File nifiLibDir){
String nifiVersion = getNiFiVersionFromProperties(nifiConfDir)
if(StringUtils.isEmpty(nifiVersion)){
nifiVersion = getNiFiVersionFromNar(nifiLibDir)
}
return nifiVersion.replace("-SNAPSHOT","")
}
}

View File

@ -0,0 +1,82 @@
/*
* 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.toolkit.admin.util
import org.apache.commons.lang3.StringUtils
class Version {
private String[] versionNumber
private String delimeter
Version(String version, String delimeter) {
this.versionNumber = version.tokenize(delimeter)
this.delimeter = delimeter
}
String[] getVersionNumber() {
return versionNumber
}
void setVersionNumber(String[] versionNumber) {
this.versionNumber = versionNumber
}
String getDelimeter() {
return delimeter
}
void setDelimeter(String delimeter) {
this.delimeter = delimeter
}
boolean equals(o) {
if (this.is(o)) return true
if (getClass() != o.class) return false
Version version = (Version) o
if (!Arrays.equals(versionNumber, version.versionNumber)) return false
return true
}
int hashCode() {
return (versionNumber != null ? Arrays.hashCode(versionNumber) : 0)
}
public final static Comparator<Version> VERSION_COMPARATOR = new Comparator<Version>() {
@Override
int compare(Version o1, Version o2) {
String[] o1V = o1.versionNumber
String[] o2V = o2.versionNumber
for(int i = 0; i < o1V.length; i++) {
Integer val1 = Integer.parseInt(o1V[i])
Integer val2 = Integer.parseInt(o2V[i])
if (val1.compareTo(val2) != 0) {
return val1.compareTo(val2)
}
}
return 0
}
}
@Override
public String toString() {
StringUtils.join(versionNumber,delimeter)
}
}

View File

@ -0,0 +1,247 @@
/*
* 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.toolkit.admin.client
import org.apache.commons.lang3.SystemUtils
import org.apache.nifi.properties.NiFiPropertiesLoader
import org.apache.nifi.security.util.CertificateUtils
import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandalone
import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandaloneCommandLine
import org.apache.nifi.util.NiFiProperties
import org.bouncycastle.asn1.x500.X500Name
import org.bouncycastle.asn1.x500.X500NameBuilder
import org.bouncycastle.asn1.x500.style.BCStyle
import org.bouncycastle.asn1.x509.Extension
import org.bouncycastle.asn1.x509.Extensions
import org.bouncycastle.asn1.x509.ExtensionsGenerator
import org.bouncycastle.asn1.x509.GeneralName
import org.bouncycastle.asn1.x509.GeneralNames
import org.bouncycastle.asn1.x509.SubjectPublicKeyInfo
import org.bouncycastle.cert.X509CertificateHolder
import org.bouncycastle.cert.X509v3CertificateBuilder
import org.bouncycastle.cert.jcajce.JcaX509CertificateConverter
import org.bouncycastle.jce.provider.BouncyCastleProvider
import org.bouncycastle.operator.ContentSigner
import org.bouncycastle.operator.OperatorCreationException
import org.bouncycastle.operator.jcajce.JcaContentSignerBuilder
import spock.lang.Specification
import javax.net.ssl.SSLSession
import java.nio.file.Files
import java.nio.file.attribute.PosixFilePermission
import java.security.InvalidKeyException
import java.security.KeyPair
import java.security.KeyPairGenerator
import java.security.NoSuchAlgorithmException
import java.security.NoSuchProviderException
import java.security.SignatureException
import java.security.cert.Certificate
import java.security.cert.CertificateException
import java.security.cert.X509Certificate
import java.util.concurrent.TimeUnit
class NiFiClientFactorySpec extends Specification {
private static final int KEY_SIZE = 2048
private static final String SIGNATURE_ALGORITHM = "SHA256withRSA"
private static final int DAYS_IN_YEAR = 365
private static final String ISSUER_DN = "CN=NiFi Test CA,OU=Security,O=Apache,ST=CA,C=US"
def "get client for unsecure nifi"(){
given:
def NiFiProperties niFiProperties = Mock NiFiProperties
def clientFactory = new NiFiClientFactory()
when:
def client = clientFactory.getClient(niFiProperties,"src/test/resources/notify")
then:
client
}
def "get client for secured nifi"(){
given:
def File tmpDir = setupTmpDir()
def File testDir = new File("target/tmp/keys")
def toolkitCommandLine = ["-O", "-o",testDir.absolutePath,"-n","localhost","-C", "CN=user1","-S", "badKeyPass", "-K", "badKeyPass", "-P", "badTrustPass"]
TlsToolkitStandaloneCommandLine tlsToolkitStandaloneCommandLine = new TlsToolkitStandaloneCommandLine()
tlsToolkitStandaloneCommandLine.parse(toolkitCommandLine as String[])
new TlsToolkitStandalone().createNifiKeystoresAndTrustStores(tlsToolkitStandaloneCommandLine.createConfig())
def bootstrapConfFile = "src/test/resources/notify/conf/bootstrap.conf"
def nifiPropertiesFile = "src/test/resources/notify/conf/nifi-secured.properties"
def key = NiFiPropertiesLoader.extractKeyFromBootstrapFile(bootstrapConfFile)
def NiFiProperties niFiProperties = NiFiPropertiesLoader.withKey(key).load(nifiPropertiesFile)
def clientFactory = new NiFiClientFactory()
when:
def client = clientFactory.getClient(niFiProperties,"src/test/resources/notify")
then:
client
cleanup:
tmpDir.deleteDir()
}
def "should verify CN in certificate based on subjectDN"(){
given:
final String EXPECTED_DN = "CN=client.nifi.apache.org,OU=Security,O=Apache,ST=CA,C=US"
Certificate[] certificateChain = generateCertificateChain(EXPECTED_DN,ISSUER_DN)
def mockSession = Mock(SSLSession)
NiFiClientFactory.NiFiHostnameVerifier verifier = new NiFiClientFactory.NiFiHostnameVerifier()
mockSession.getPeerCertificates() >> certificateChain
when:
def verified = verifier.verify("client.nifi.apache.org",mockSession)
then:
verified
}
def "should not verify based on no certificate chain"(){
given:
final String EXPECTED_DN = "CN=client.nifi.apache.org, OU=Security, O=Apache, ST=CA, C=US"
Certificate[] certificateChain = [] as Certificate[]
def mockSession = Mock(SSLSession)
NiFiClientFactory.NiFiHostnameVerifier verifier = new NiFiClientFactory.NiFiHostnameVerifier()
mockSession.getPeerCertificates() >> certificateChain
when:
def notVerified = !verifier.verify("client.nifi.apache.org",mockSession)
then:
notVerified
}
def "should not verify based on multiple CN values"(){
given:
final KeyPair issuerKeyPair = generateKeyPair()
KeyPair keyPair = generateKeyPair()
final X509Certificate issuerCertificate = CertificateUtils.generateSelfSignedX509Certificate(issuerKeyPair,ISSUER_DN, SIGNATURE_ALGORITHM, DAYS_IN_YEAR)
ContentSigner sigGen = new JcaContentSignerBuilder(SIGNATURE_ALGORITHM).setProvider(BouncyCastleProvider.PROVIDER_NAME).build(issuerKeyPair.getPrivate());
SubjectPublicKeyInfo subPubKeyInfo = SubjectPublicKeyInfo.getInstance(keyPair.public.getEncoded());
Date startDate = new Date();
Date endDate = new Date(startDate.getTime() + TimeUnit.DAYS.toMillis(DAYS_IN_YEAR));
def X500NameBuilder nameBuilder = new X500NameBuilder(BCStyle.INSTANCE)
nameBuilder.addRDN(BCStyle.CN,"client.nifi.apache.org,nifi.apache.org")
def name = nameBuilder.build()
X509v3CertificateBuilder certBuilder = new X509v3CertificateBuilder(new X500Name(issuerCertificate.getSubjectX500Principal().getName()),
BigInteger.valueOf(System.currentTimeMillis()), startDate, endDate, name,
subPubKeyInfo);
X509CertificateHolder certificateHolder = certBuilder.build(sigGen);
Certificate certificate = new JcaX509CertificateConverter().setProvider(BouncyCastleProvider.PROVIDER_NAME).getCertificate(certificateHolder);
Certificate[] certificateChain = [certificate,issuerCertificate] as Certificate[]
def mockSession = Mock(SSLSession)
NiFiClientFactory.NiFiHostnameVerifier verifier = new NiFiClientFactory.NiFiHostnameVerifier()
mockSession.getPeerCertificates() >> certificateChain
when:
def notVerified = !verifier.verify("client.nifi.apache.org",mockSession)
then:
notVerified
}
def "should verify appropriately CN in certificate based on SAN"(){
given:
final List<String> SANS = ["127.0.0.1", "nifi.apache.org"]
def gns = SANS.collect { String san ->
new GeneralName(GeneralName.dNSName, san)
}
def generalNames = new GeneralNames(gns as GeneralName[])
ExtensionsGenerator extensionsGenerator = new ExtensionsGenerator()
extensionsGenerator.addExtension(Extension.subjectAlternativeName, false, generalNames)
Extensions extensions = extensionsGenerator.generate()
final String EXPECTED_DN = "CN=client.nifi.apache.org,OU=Security,O=Apache,ST=CA,C=US"
final KeyPair issuerKeyPair = generateKeyPair()
final X509Certificate issuerCertificate = CertificateUtils.generateSelfSignedX509Certificate(issuerKeyPair,ISSUER_DN, SIGNATURE_ALGORITHM, DAYS_IN_YEAR)
final X509Certificate certificate = generateIssuedCertificate(EXPECTED_DN, issuerCertificate,extensions, issuerKeyPair)
Certificate[] certificateChain = [certificate, issuerCertificate] as X509Certificate[]
def mockSession = Mock(SSLSession)
NiFiClientFactory.NiFiHostnameVerifier verifier = new NiFiClientFactory.NiFiHostnameVerifier()
mockSession.getPeerCertificates() >> certificateChain
when:
def verified = verifier.verify("nifi.apache.org",mockSession)
def notVerified = !verifier.verify("fake.apache.org",mockSession)
then:
verified
notVerified
}
def KeyPair generateKeyPair() throws NoSuchAlgorithmException {
KeyPairGenerator keyPairGenerator = KeyPairGenerator.getInstance("RSA")
keyPairGenerator.initialize(KEY_SIZE)
return keyPairGenerator.generateKeyPair()
}
def X509Certificate generateIssuedCertificate(String dn, X509Certificate issuer,Extensions extensions, KeyPair issuerKey) throws IOException, NoSuchAlgorithmException, CertificateException, NoSuchProviderException, SignatureException, InvalidKeyException, OperatorCreationException {
KeyPair keyPair = generateKeyPair()
return CertificateUtils.generateIssuedCertificate(dn, keyPair.getPublic(),extensions, issuer, issuerKey, SIGNATURE_ALGORITHM, DAYS_IN_YEAR)
}
def X509Certificate[] generateCertificateChain(String dn,String issuerDn) {
final KeyPair issuerKeyPair = generateKeyPair()
final X509Certificate issuerCertificate = CertificateUtils.generateSelfSignedX509Certificate(issuerKeyPair, issuerDn, SIGNATURE_ALGORITHM, DAYS_IN_YEAR)
final X509Certificate certificate = generateIssuedCertificate(dn, issuerCertificate,null, issuerKeyPair)
[certificate, issuerCertificate] as X509Certificate[]
}
def setFilePermissions(File file, List<PosixFilePermission> permissions = []) {
if (SystemUtils.IS_OS_WINDOWS) {
file?.setReadable(permissions.contains(PosixFilePermission.OWNER_READ))
file?.setWritable(permissions.contains(PosixFilePermission.OWNER_WRITE))
file?.setExecutable(permissions.contains(PosixFilePermission.OWNER_EXECUTE))
} else {
Files.setPosixFilePermissions(file?.toPath(), permissions as Set)
}
}
def setupTmpDir(String tmpDirPath = "target/tmp/") {
File tmpDir = new File(tmpDirPath)
tmpDir.mkdirs()
setFilePermissions(tmpDir, [PosixFilePermission.OWNER_READ, PosixFilePermission.OWNER_WRITE, PosixFilePermission.OWNER_EXECUTE,
PosixFilePermission.GROUP_READ, PosixFilePermission.GROUP_WRITE, PosixFilePermission.GROUP_EXECUTE,
PosixFilePermission.OTHERS_READ, PosixFilePermission.OTHERS_WRITE, PosixFilePermission.OTHERS_EXECUTE])
tmpDir
}
}

View File

@ -0,0 +1,109 @@
/*
* 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.toolkit.admin.client
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.ClientResponse
import com.sun.jersey.api.client.WebResource
import org.apache.nifi.util.NiFiProperties
import org.apache.nifi.web.api.entity.ClusterEntity
import org.junit.Rule
import org.junit.contrib.java.lang.system.ExpectedSystemExit
import org.junit.contrib.java.lang.system.SystemOutRule
import spock.lang.Specification
import javax.ws.rs.core.Response
class NiFiClientUtilSpec extends Specification{
@Rule
public final ExpectedSystemExit exit = ExpectedSystemExit.none()
@Rule
public final SystemOutRule systemOutRule = new SystemOutRule().enableLog()
def "build unsecure url successfully"(){
given:
def NiFiProperties niFiProperties = Mock NiFiProperties
when:
def url = NiFiClientUtil.getUrl(niFiProperties,"/nifi-api/controller/cluster/nodes/1")
then:
3 * niFiProperties.getProperty(_)
url == "http://localhost:8080/nifi-api/controller/cluster/nodes/1"
}
def "get cluster info successfully"(){
given:
def Client client = Mock Client
def NiFiProperties niFiProperties = Mock NiFiProperties
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def ClusterEntity clusterEntity = Mock ClusterEntity
when:
def entity = NiFiClientUtil.getCluster(client, niFiProperties, [])
then:
3 * niFiProperties.getProperty(_)
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.get(_) >> response
1 * response.getStatus() >> 200
1 * response.getEntity(ClusterEntity.class) >> clusterEntity
entity == clusterEntity
}
def "get cluster info fails"(){
given:
def Client client = Mock Client
def NiFiProperties niFiProperties = Mock NiFiProperties
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def Response.StatusType statusType = Mock Response.StatusType
when:
NiFiClientUtil.getCluster(client, niFiProperties, [])
then:
3 * niFiProperties.getProperty(_)
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.get(_) >> response
1 * response.getStatus() >> 500
1 * response.getStatusInfo() >> statusType
1 * statusType.getReasonPhrase() >> "Only a node connected to a cluster can process the request."
def e = thrown(RuntimeException)
e.message == "Unable to obtain cluster information"
}
}

View File

@ -0,0 +1,414 @@
/*
* 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.toolkit.admin.nodemanager
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.ClientResponse
import com.sun.jersey.api.client.WebResource
import org.apache.commons.cli.ParseException
import org.apache.nifi.toolkit.admin.client.ClientFactory
import org.apache.nifi.util.NiFiProperties
import org.apache.nifi.web.api.dto.ClusterDTO
import org.apache.nifi.web.api.dto.NodeDTO
import org.apache.nifi.web.api.entity.ClusterEntity
import org.apache.nifi.web.api.entity.NodeEntity
import org.junit.Rule
import org.junit.contrib.java.lang.system.ExpectedSystemExit
import org.junit.contrib.java.lang.system.SystemOutRule
import spock.lang.Specification
class NodeManagerToolSpec extends Specification{
@Rule
public final ExpectedSystemExit exit = ExpectedSystemExit.none()
@Rule
public final SystemOutRule systemOutRule = new SystemOutRule().enableLog()
def "print help and usage info"() {
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NodeManagerTool()
when:
config.parse(clientFactory,["-h"] as String[])
then:
systemOutRule.getLog().contains("usage: org.apache.nifi.toolkit.admin.nodemanager.NodeManagerTool")
}
def "throws exception missing bootstrap conf flag"() {
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NodeManagerTool()
when:
config.parse(clientFactory,["-d", "/install/nifi"] as String[])
then:
def e = thrown(ParseException)
e.message == "Missing -b option"
}
def "throws exception missing directory"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NodeManagerTool()
when:
config.parse(clientFactory,["-b","src/test/resources/notify/conf/bootstrap.conf"] as String[])
then:
def e = thrown(ParseException)
e.message == "Missing -d option"
}
def "throws exception missing operation"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NodeManagerTool()
when:
config.parse(clientFactory,["-b","src/test/resources/notify/conf/bootstrap.conf","-d", "/install/nifi"] as String[])
then:
def e = thrown(ParseException)
e.message == "Missing -o option"
}
def "throws exception invalid operation"(){
given:
def NiFiProperties niFiProperties = Mock NiFiProperties
def ClientFactory clientFactory = Mock ClientFactory
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def ClusterEntity clusterEntity = Mock ClusterEntity
def ClusterDTO clusterDTO = Mock ClusterDTO
def NodeDTO nodeDTO = new NodeDTO()
nodeDTO.address = "localhost"
nodeDTO.nodeId = "1"
nodeDTO.status = "CONNECTED"
nodeDTO.apiPort = 8080
def List<NodeDTO> nodeDTOs = [nodeDTO]
def NodeEntity nodeEntity = new NodeEntity()
nodeEntity.node = nodeDTO
def config = new NodeManagerTool()
niFiProperties.getProperty(_) >> "localhost"
clientFactory.getClient(_,_) >> client
client.resource(_ as String) >> resource
resource.type(_) >> builder
builder.get(ClientResponse.class) >> response
builder.put(_,_) >> response
builder.delete(ClientResponse.class,_) >> response
response.getStatus() >> 200
response.getEntity(ClusterEntity.class) >> clusterEntity
response.getEntity(NodeEntity.class) >> nodeEntity
clusterEntity.getCluster() >> clusterDTO
clusterDTO.getNodes() >> nodeDTOs
nodeDTO.address >> "localhost"
when:
config.parse(clientFactory,["-b","src/test/resources/notify/conf/bootstrap.conf","-d","/install/nifi","-o","fake"] as String[])
then:
def e = thrown(ParseException)
e.message == "Invalid operation provided: fake"
}
def "get node info successfully"(){
given:
def NiFiProperties niFiProperties = Mock NiFiProperties
def ClusterEntity clusterEntity = Mock ClusterEntity
def ClusterDTO clusterDTO = Mock ClusterDTO
def NodeDTO nodeDTO = new NodeDTO()
nodeDTO.address = "1"
def List<NodeDTO> nodeDTOs = [nodeDTO]
def config = new NodeManagerTool()
when:
def entity = config.getCurrentNode(clusterEntity,niFiProperties)
then:
1 * clusterEntity.getCluster() >> clusterDTO
1 * clusterDTO.getNodes() >> nodeDTOs
2 * niFiProperties.getProperty(_) >> "1"
entity == nodeDTO
}
def "delete node successfully"(){
given:
def String url = "http://locahost:8080/nifi-api/controller"
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def config = new NodeManagerTool()
when:
config.deleteNode(url,client)
then:
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.delete(_) >> response
1 * response.getStatus() >> 200
}
def "delete node failed"(){
given:
def String url = "http://locahost:8080/nifi-api/controller"
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def config = new NodeManagerTool()
when:
config.deleteNode(url,client)
then:
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.delete(_) >> response
2 * response.getStatus() >> 403
def e = thrown(RuntimeException)
e.message == "Failed with HTTP error code: 403"
}
def "update node successfully"(){
given:
def String url = "http://locahost:8080/nifi-api/controller"
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def NodeDTO nodeDTO = new NodeDTO()
def NodeEntity nodeEntity = Mock NodeEntity
def config = new NodeManagerTool()
when:
def entity = config.updateNode(url,client,nodeDTO,NodeManagerTool.STATUS.DISCONNECTING)
then:
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.put(_,_) >> response
1 * response.getStatus() >> 200
1 * response.getEntity(NodeEntity.class) >> nodeEntity
entity == nodeEntity
}
def "update node fails"(){
given:
def String url = "http://locahost:8080/nifi-api/controller"
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def NodeDTO nodeDTO = new NodeDTO()
def config = new NodeManagerTool()
when:
config.updateNode(url,client,nodeDTO,NodeManagerTool.STATUS.DISCONNECTING)
then:
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.put(_,_) >> response
2 * response.getStatus() >> 403
def e = thrown(RuntimeException)
e.message == "Failed with HTTP error code: 403"
}
def "disconnect node successfully"(){
setup:
def NiFiProperties niFiProperties = Mock NiFiProperties
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def ClusterEntity clusterEntity = Mock ClusterEntity
def ClusterDTO clusterDTO = Mock ClusterDTO
def NodeDTO nodeDTO = new NodeDTO()
nodeDTO.address = "localhost"
nodeDTO.nodeId = "1"
nodeDTO.status = "CONNECTED"
def List<NodeDTO> nodeDTOs = [nodeDTO]
def NodeEntity nodeEntity = new NodeEntity()
nodeEntity.node = nodeDTO
def config = new NodeManagerTool()
niFiProperties.getProperty(_) >> "localhost"
client.resource(_ as String) >> resource
resource.type(_) >> builder
builder.get(ClientResponse.class) >> response
builder.put(_,_) >> response
response.getStatus() >> 200
response.getEntity(ClusterEntity.class) >> clusterEntity
response.getEntity(NodeEntity.class) >> nodeEntity
clusterEntity.getCluster() >> clusterDTO
clusterDTO.getNodes() >> nodeDTOs
nodeDTO.address >> "localhost"
expect:
config.disconnectNode(client, niFiProperties,["http://localhost:8080"])
}
def "connect node successfully"(){
setup:
def NiFiProperties niFiProperties = Mock NiFiProperties
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def ClusterEntity clusterEntity = Mock ClusterEntity
def ClusterDTO clusterDTO = Mock ClusterDTO
def NodeDTO nodeDTO = new NodeDTO()
nodeDTO.address = "localhost"
nodeDTO.nodeId = "1"
nodeDTO.status = "DISCONNECTED"
def List<NodeDTO> nodeDTOs = [nodeDTO]
def NodeEntity nodeEntity = new NodeEntity()
nodeEntity.node = nodeDTO
def config = new NodeManagerTool()
niFiProperties.getProperty(_) >> "localhost"
client.resource(_ as String) >> resource
resource.type(_) >> builder
builder.get(ClientResponse.class) >> response
builder.put(_,_) >> response
response.getStatus() >> 200
response.getEntity(ClusterEntity.class) >> clusterEntity
response.getEntity(NodeEntity.class) >> nodeEntity
clusterEntity.getCluster() >> clusterDTO
clusterDTO.getNodes() >> nodeDTOs
nodeDTO.address >> "localhost"
expect:
config.connectNode(client, niFiProperties,["http://localhost:8080"])
}
def "remove node successfully"(){
setup:
def NiFiProperties niFiProperties = Mock NiFiProperties
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def ClusterEntity clusterEntity = Mock ClusterEntity
def ClusterDTO clusterDTO = Mock ClusterDTO
def NodeDTO nodeDTO = new NodeDTO()
nodeDTO.address = "localhost"
nodeDTO.nodeId = "1"
nodeDTO.status = "CONNECTED"
def List<NodeDTO> nodeDTOs = [nodeDTO]
def NodeEntity nodeEntity = new NodeEntity()
nodeEntity.node = nodeDTO
def config = new NodeManagerTool()
niFiProperties.getProperty(_) >> "localhost"
client.resource(_ as String) >> resource
resource.type(_) >> builder
builder.get(ClientResponse.class) >> response
builder.put(_,_) >> response
builder.delete(ClientResponse.class,_) >> response
response.getStatus() >> 200
response.getEntity(ClusterEntity.class) >> clusterEntity
response.getEntity(NodeEntity.class) >> nodeEntity
clusterEntity.getCluster() >> clusterDTO
clusterDTO.getNodes() >> nodeDTOs
nodeDTO.address >> "localhost"
expect:
config.removeNode(client, niFiProperties,["http://localhost:8080"])
}
def "parse args and delete node"(){
setup:
def NiFiProperties niFiProperties = Mock NiFiProperties
def ClientFactory clientFactory = Mock ClientFactory
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def ClusterEntity clusterEntity = Mock ClusterEntity
def ClusterDTO clusterDTO = Mock ClusterDTO
def NodeDTO nodeDTO = new NodeDTO()
nodeDTO.address = "localhost"
nodeDTO.nodeId = "1"
nodeDTO.status = "CONNECTED"
def List<NodeDTO> nodeDTOs = [nodeDTO]
def NodeEntity nodeEntity = new NodeEntity()
nodeEntity.node = nodeDTO
def config = new NodeManagerTool()
niFiProperties.getProperty(_) >> "localhost"
clientFactory.getClient(_,_) >> client
client.resource(_ as String) >> resource
resource.type(_) >> builder
builder.get(ClientResponse.class) >> response
builder.put(_,_) >> response
builder.delete(ClientResponse.class,_) >> response
response.getStatus() >> 200
response.getEntity(ClusterEntity.class) >> clusterEntity
response.getEntity(NodeEntity.class) >> nodeEntity
clusterEntity.getCluster() >> clusterDTO
clusterDTO.getNodes() >> nodeDTOs
nodeDTO.address >> "localhost"
expect:
config.parse(clientFactory,["-b","src/test/resources/notify/conf/bootstrap.conf","-d","/bogus/nifi/dir","-o","remove","-u","http://localhost:8080,http://localhost1:8080"] as String[])
}
}

View File

@ -0,0 +1,171 @@
/*
* 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.toolkit.admin.notify
import com.sun.jersey.api.client.Client
import com.sun.jersey.api.client.ClientResponse
import com.sun.jersey.api.client.WebResource
import org.apache.commons.cli.ParseException
import org.apache.nifi.toolkit.admin.client.ClientFactory
import org.junit.Rule
import org.junit.contrib.java.lang.system.ExpectedSystemExit
import org.junit.contrib.java.lang.system.SystemOutRule
import spock.lang.Specification
class NotificationToolSpec extends Specification{
@Rule
public final ExpectedSystemExit exit = ExpectedSystemExit.none()
@Rule
public final SystemOutRule systemOutRule = new SystemOutRule().enableLog()
def "print help and usage info"() {
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NotificationTool()
when:
config.parse(clientFactory,["-h"] as String[])
then:
systemOutRule.getLog().contains("usage: org.apache.nifi.toolkit.admin.notify.NotificationTool")
}
def "throws exception missing bootstrap conf flag"() {
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NotificationTool()
when:
config.parse(clientFactory,["-d", "/missing/bootstrap/conf"] as String[])
then:
def e = thrown(ParseException)
e.message == "Missing -b option"
}
def "throws exception missing message"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NotificationTool()
when:
config.parse(clientFactory,["-b","/tmp/fake/upgrade/conf","-v","-d","/bogus/nifi/dir"] as String[])
then:
def e = thrown(ParseException)
e.message == "Missing -m option"
}
def "throws exception missing directory"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def config = new NotificationTool()
when:
config.parse(clientFactory,["-b","src/test/resources/notify/conf/bootstrap.conf","-m","shutting down in 30 seconds"] as String[])
then:
def e = thrown(ParseException)
e.message == "Missing -d option"
}
def "send cluster message successfully"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def config = new NotificationTool()
when:
config.notifyCluster(clientFactory,"src/test/resources/notify/conf/nifi.properties","src/test/resources/notify/conf/bootstrap.conf","/bogus/nifi/dir","shutting down in 30 seconds","WARN")
then:
1 * clientFactory.getClient(_,_) >> client
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.post(_,_) >> response
1 * response.getStatus() >> 200
}
def "cluster message failed"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def config = new NotificationTool()
when:
config.notifyCluster(clientFactory,"src/test/resources/notify/conf/nifi.properties","src/test/resources/notify/conf/bootstrap.conf","/bogus/nifi/dir","shutting down in 30 seconds","WARN")
then:
1 * clientFactory.getClient(_,_) >> client
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.post(_,_) >> response
1 * response.getStatus() >> 403
def e = thrown(RuntimeException)
e.message == "Failed with HTTP error code: 403"
}
def "parse comment and send cluster message successfully"(){
given:
def ClientFactory clientFactory = Mock ClientFactory
def Client client = Mock Client
def WebResource resource = Mock WebResource
def WebResource.Builder builder = Mock WebResource.Builder
def ClientResponse response = Mock ClientResponse
def config = new NotificationTool()
when:
config.parse(clientFactory,["-b","src/test/resources/notify/conf/bootstrap.conf","-d","/bogus/nifi/dir","-m","shutting down in 30 seconds","-l","ERROR"] as String[])
then:
1 * clientFactory.getClient(_,_) >> client
1 * client.resource(_ as String) >> resource
1 * resource.type(_) >> builder
1 * builder.post(_,_) >> response
1 * response.getStatus() >> 200
}
}

View File

@ -0,0 +1,54 @@
/*
* 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.toolkit.admin.util
import spock.lang.Specification
class AdminUtilSpec extends Specification{
def "get nifi version with version in properties"(){
setup:
def nifiConfDir = new File("src/test/resources/conf")
def nifiLibDir = new File("src/test/resources/lib")
when:
def version = AdminUtil.getNiFiVersion(nifiConfDir,nifiLibDir)
then:
version == "1.1.0"
}
def "get nifi version with version in nar"(){
setup:
def nifiConfDir = new File("src/test/resources/upgrade/conf")
def nifiLibDir = new File("src/test/resources/lib")
when:
def version = AdminUtil.getNiFiVersion(nifiConfDir,nifiLibDir)
then:
version == "1.2.0"
}
}

View File

@ -0,0 +1,32 @@
#
# 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.
#
# Java command to use when running NiFi
java=java
# Username to use when running NiFi. This value will be ignored on Windows.
run.as=
# Configure where NiFi's lib and conf directories live
lib.dir=./lib
conf.dir=./conf
# How long to wait after telling NiFi to shutdown before explicitly killing the Process
graceful.shutdown.seconds=20
# Disable JSR 199 so that we can use JSP's without running a JDK
java.arg.1=-Dorg.apache.jasper.compiler.disablejsr199=true

View File

@ -0,0 +1,112 @@
<?xml version="1.0" encoding="UTF-8" standalone="yes"?>
<!--
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.
-->
<!--
This file lists the login identity providers to use when running securely. In order
to use a specific provider it must be configured here and it's identifier
must be specified in the nifi.properties file.
-->
<loginIdentityProviders>
<!--
Identity Provider for users logging in with username/password against an LDAP server.
'Authentication Strategy' - How the connection to the LDAP server is authenticated. Possible
values are ANONYMOUS, SIMPLE, LDAPS, or START_TLS.
'Manager DN' - The DN of the manager that is used to bind to the LDAP server to search for users.
'Manager Password' - The password of the manager that is used to bind to the LDAP server to
search for users.
'TLS - Keystore' - Path to the Keystore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Keystore Password' - Password for the Keystore that is used when connecting to LDAP
using LDAPS or START_TLS.
'TLS - Keystore Type' - Type of the Keystore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Truststore' - Path to the Truststore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Truststore Password' - Password for the Truststore that is used when connecting to
LDAP using LDAPS or START_TLS.
'TLS - Truststore Type' - Type of the Truststore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Client Auth' - Client authentication policy when connecting to LDAP using LDAPS or START_TLS.
Possible values are REQUIRED, WANT, NONE.
'TLS - Protocol' - Protocol to use when connecting to LDAP using LDAPS or START_TLS. (i.e. TLS,
TLSv1.1, TLSv1.2, etc).
'TLS - Shutdown Gracefully' - Specifies whether the TLS should be shut down gracefully
before the target context is closed. Defaults to false.
'Referral Strategy' - Strategy for handling referrals. Possible values are FOLLOW, IGNORE, THROW.
'Connect Timeout' - Duration of connect timeout. (i.e. 10 secs).
'Read Timeout' - Duration of read timeout. (i.e. 10 secs).
'Url' - Space-separated list of URLs of the LDAP servers (i.e. ldap://<hostname>:<port>).
'User Search Base' - Base DN for searching for users (i.e. CN=Users,DC=example,DC=com).
'User Search Filter' - Filter for searching for users against the 'User Search Base'.
(i.e. sAMAccountName={0}). The user specified name is inserted into '{0}'.
'Identity Strategy' - Strategy to identify users. Possible values are USE_DN and USE_USERNAME.
The default functionality if this property is missing is USE_DN in order to retain
backward compatibility. USE_DN will use the full DN of the user entry if possible.
USE_USERNAME will use the username the user logged in with.
'Authentication Expiration' - The duration of how long the user authentication is valid
for. If the user never logs out, they will be required to log back in following
this duration.
-->
<!-- To enable the ldap-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>ldap-provider</identifier>
<class>org.apache.nifi.ldap.LdapProvider</class>
<property name="Authentication Strategy">START_TLS</property>
<property name="Manager DN"></property>
<property name="Manager Password"></property>
<property name="TLS - Keystore"></property>
<property name="TLS - Keystore Password"></property>
<property name="TLS - Keystore Type"></property>
<property name="TLS - Truststore"></property>
<property name="TLS - Truststore Password"></property>
<property name="TLS - Truststore Type"></property>
<property name="TLS - Client Auth"></property>
<property name="TLS - Protocol"></property>
<property name="TLS - Shutdown Gracefully"></property>
<property name="Referral Strategy">FOLLOW</property>
<property name="Connect Timeout">10 secs</property>
<property name="Read Timeout">10 secs</property>
<property name="Url"></property>
<property name="User Search Base"></property>
<property name="User Search Filter"></property>
<property name="Identity Strategy">USE_DN</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the ldap-provider remove 2 lines. This is 2 of 2. -->
<!--
Identity Provider for users logging in with username/password against a Kerberos KDC server.
'Default Realm' - Default realm to provide when user enters incomplete user principal (i.e. NIFI.APACHE.ORG).
'Authentication Expiration' - The duration of how long the user authentication is valid for. If the user never logs out, they will be required to log back in following this duration.
-->
<!-- To enable the kerberos-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>kerberos-provider</identifier>
<class>org.apache.nifi.kerberos.KerberosProvider</class>
<property name="Default Realm">NIFI.APACHE.ORG</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the kerberos-provider remove 2 lines. This is 2 of 2. -->
</loginIdentityProviders>

View File

@ -0,0 +1,28 @@
#
# 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.
#
nifi.version=1.1.0
nifi.cluster.is.node=true
nifi.cluster.node.address=localhost
nifi.cluster.node.protocol.port=8300
nifi.cluster.node.protocol.threads=2
nifi.cluster.node.event.history.size=
nifi.cluster.node.connection.timeout=
nifi.cluster.node.read.timeout=30
nifi.cluster.firewall.file=
nifi.cluster.flow.election.max.wait.time=1
nifi.cluster.flow.election.max.candidates=
nifi.fluster.an.old.variable=true

View File

@ -0,0 +1,32 @@
#
# 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.
#
# Java command to use when running NiFi
java=java
# Username to use when running NiFi. This value will be ignored on Windows.
run.as=
# Configure where NiFi's lib and conf directories live
lib.dir=./lib
conf.dir=target/tmp/conf
# How long to wait after telling NiFi to shutdown before explicitly killing the Process
graceful.shutdown.seconds=20
# Disable JSR 199 so that we can use JSP's without running a JDK
java.arg.1=-Dorg.apache.jasper.compiler.disablejsr199=true

View File

@ -0,0 +1,112 @@
<?xml version="1.0" encoding="UTF-8" standalone="yes"?>
<!--
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.
-->
<!--
This file lists the login identity providers to use when running securely. In order
to use a specific provider it must be configured here and it's identifier
must be specified in the nifi.properties file.
-->
<loginIdentityProviders>
<!--
Identity Provider for users logging in with username/password against an LDAP server.
'Authentication Strategy' - How the connection to the LDAP server is authenticated. Possible
values are ANONYMOUS, SIMPLE, LDAPS, or START_TLS.
'Manager DN' - The DN of the manager that is used to bind to the LDAP server to search for users.
'Manager Password' - The password of the manager that is used to bind to the LDAP server to
search for users.
'TLS - Keystore' - Path to the Keystore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Keystore Password' - Password for the Keystore that is used when connecting to LDAP
using LDAPS or START_TLS.
'TLS - Keystore Type' - Type of the Keystore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Truststore' - Path to the Truststore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Truststore Password' - Password for the Truststore that is used when connecting to
LDAP using LDAPS or START_TLS.
'TLS - Truststore Type' - Type of the Truststore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Client Auth' - Client authentication policy when connecting to LDAP using LDAPS or START_TLS.
Possible values are REQUIRED, WANT, NONE.
'TLS - Protocol' - Protocol to use when connecting to LDAP using LDAPS or START_TLS. (i.e. TLS,
TLSv1.1, TLSv1.2, etc).
'TLS - Shutdown Gracefully' - Specifies whether the TLS should be shut down gracefully
before the target context is closed. Defaults to false.
'Referral Strategy' - Strategy for handling referrals. Possible values are FOLLOW, IGNORE, THROW.
'Connect Timeout' - Duration of connect timeout. (i.e. 10 secs).
'Read Timeout' - Duration of read timeout. (i.e. 10 secs).
'Url' - Space-separated list of URLs of the LDAP servers (i.e. ldap://<hostname>:<port>).
'User Search Base' - Base DN for searching for users (i.e. CN=Users,DC=example,DC=com).
'User Search Filter' - Filter for searching for users against the 'User Search Base'.
(i.e. sAMAccountName={0}). The user specified name is inserted into '{0}'.
'Identity Strategy' - Strategy to identify users. Possible values are USE_DN and USE_USERNAME.
The default functionality if this property is missing is USE_DN in order to retain
backward compatibility. USE_DN will use the full DN of the user entry if possible.
USE_USERNAME will use the username the user logged in with.
'Authentication Expiration' - The duration of how long the user authentication is valid
for. If the user never logs out, they will be required to log back in following
this duration.
-->
<!-- To enable the ldap-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>ldap-provider</identifier>
<class>org.apache.nifi.ldap.LdapProvider</class>
<property name="Authentication Strategy">START_TLS</property>
<property name="Manager DN"></property>
<property name="Manager Password"></property>
<property name="TLS - Keystore"></property>
<property name="TLS - Keystore Password"></property>
<property name="TLS - Keystore Type"></property>
<property name="TLS - Truststore"></property>
<property name="TLS - Truststore Password"></property>
<property name="TLS - Truststore Type"></property>
<property name="TLS - Client Auth"></property>
<property name="TLS - Protocol"></property>
<property name="TLS - Shutdown Gracefully"></property>
<property name="Referral Strategy">FOLLOW</property>
<property name="Connect Timeout">10 secs</property>
<property name="Read Timeout">10 secs</property>
<property name="Url"></property>
<property name="User Search Base"></property>
<property name="User Search Filter"></property>
<property name="Identity Strategy">USE_DN</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the ldap-provider remove 2 lines. This is 2 of 2. -->
<!--
Identity Provider for users logging in with username/password against a Kerberos KDC server.
'Default Realm' - Default realm to provide when user enters incomplete user principal (i.e. NIFI.APACHE.ORG).
'Authentication Expiration' - The duration of how long the user authentication is valid for. If the user never logs out, they will be required to log back in following this duration.
-->
<!-- To enable the kerberos-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>kerberos-provider</identifier>
<class>org.apache.nifi.kerberos.KerberosProvider</class>
<property name="Default Realm">NIFI.APACHE.ORG</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the kerberos-provider remove 2 lines. This is 2 of 2. -->
</loginIdentityProviders>

View File

@ -0,0 +1,28 @@
#
# 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.
#
nifi.version=1.1.0
nifi.cluster.is.node=true
nifi.cluster.node.address=localhost
nifi.cluster.node.protocol.port=8300
nifi.cluster.node.protocol.threads=2
nifi.cluster.node.event.history.size=
nifi.cluster.node.connection.timeout=
nifi.cluster.node.read.timeout=30
nifi.cluster.firewall.file=
nifi.cluster.flow.election.max.wait.time=1
nifi.cluster.flow.election.max.candidates=
nifi.fluster.an.old.variable=true

View File

@ -0,0 +1,32 @@
#
# 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.
#
# Java command to use when running NiFi
java=java
# Username to use when running NiFi. This value will be ignored on Windows.
run.as=
# Configure where NiFi's lib and conf directories live
lib.dir=./lib
conf.dir=./conf
# How long to wait after telling NiFi to shutdown before explicitly killing the Process
graceful.shutdown.seconds=20
# Disable JSR 199 so that we can use JSP's without running a JDK
java.arg.1=-Dorg.apache.jasper.compiler.disablejsr199=true

View File

@ -0,0 +1,32 @@
#
# 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.
#
nifi.version=1.1.0
nifi.cluster.is.node=true
nifi.cluster.node.address=localhost
nifi.cluster.node.protocol.port=8300
nifi.cluster.node.protocol.threads=2
nifi.cluster.node.event.history.size=
nifi.cluster.node.connection.timeout=
nifi.cluster.node.read.timeout=30
nifi.cluster.firewall.file=
nifi.cluster.flow.election.max.wait.time=1
nifi.cluster.flow.election.max.candidates=
nifi.fluster.an.old.variable=true
nifi.content.repository.directory.default=./content_repository
nifi.provenance.repository.directory.default=./provenance_repository
nifi.flowfile.repository.directory=./flowfile_repository
nifi.database.directory=./database_repository

View File

@ -0,0 +1,21 @@
#
# 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.
#
# JVM memory settings
java.arg.2=-Xms512m
java.arg.3=-Xmx512m

View File

@ -0,0 +1,112 @@
<?xml version="1.0" encoding="UTF-8" standalone="yes"?>
<!--
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.
-->
<!--
This file lists the login identity providers to use when running securely. In order
to use a specific provider it must be configured here and it's identifier
must be specified in the nifi.properties file.
-->
<loginIdentityProviders>
<!--
Identity Provider for users logging in with username/password against an LDAP server.
'Authentication Strategy' - How the connection to the LDAP server is authenticated. Possible
values are ANONYMOUS, SIMPLE, LDAPS, or START_TLS.
'Manager DN' - The DN of the manager that is used to bind to the LDAP server to search for users.
'Manager Password' - The password of the manager that is used to bind to the LDAP server to
search for users.
'TLS - Keystore' - Path to the Keystore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Keystore Password' - Password for the Keystore that is used when connecting to LDAP
using LDAPS or START_TLS.
'TLS - Keystore Type' - Type of the Keystore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Truststore' - Path to the Truststore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Truststore Password' - Password for the Truststore that is used when connecting to
LDAP using LDAPS or START_TLS.
'TLS - Truststore Type' - Type of the Truststore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Client Auth' - Client authentication policy when connecting to LDAP using LDAPS or START_TLS.
Possible values are REQUIRED, WANT, NONE.
'TLS - Protocol' - Protocol to use when connecting to LDAP using LDAPS or START_TLS. (i.e. TLS,
TLSv1.1, TLSv1.2, etc).
'TLS - Shutdown Gracefully' - Specifies whether the TLS should be shut down gracefully
before the target context is closed. Defaults to false.
'Referral Strategy' - Strategy for handling referrals. Possible values are FOLLOW, IGNORE, THROW.
'Connect Timeout' - Duration of connect timeout. (i.e. 10 secs).
'Read Timeout' - Duration of read timeout. (i.e. 10 secs).
'Url' - Space-separated list of URLs of the LDAP servers (i.e. ldap://<hostname>:<port>).
'User Search Base' - Base DN for searching for users (i.e. CN=Users,DC=example,DC=com).
'User Search Filter' - Filter for searching for users against the 'User Search Base'.
(i.e. sAMAccountName={0}). The user specified name is inserted into '{0}'.
'Identity Strategy' - Strategy to identify users. Possible values are USE_DN and USE_USERNAME.
The default functionality if this property is missing is USE_DN in order to retain
backward compatibility. USE_DN will use the full DN of the user entry if possible.
USE_USERNAME will use the username the user logged in with.
'Authentication Expiration' - The duration of how long the user authentication is valid
for. If the user never logs out, they will be required to log back in following
this duration.
-->
<!-- To enable the ldap-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>ldap-provider</identifier>
<class>org.apache.nifi.ldap.LdapProvider</class>
<property name="Authentication Strategy">START_TLS</property>
<property name="Manager DN"></property>
<property name="Manager Password"></property>
<property name="TLS - Keystore"></property>
<property name="TLS - Keystore Password"></property>
<property name="TLS - Keystore Type"></property>
<property name="TLS - Truststore"></property>
<property name="TLS - Truststore Password"></property>
<property name="TLS - Truststore Type"></property>
<property name="TLS - Client Auth"></property>
<property name="TLS - Protocol"></property>
<property name="TLS - Shutdown Gracefully"></property>
<property name="Referral Strategy">FOLLOW</property>
<property name="Connect Timeout">10 secs</property>
<property name="Read Timeout">10 secs</property>
<property name="Url"></property>
<property name="User Search Base"></property>
<property name="User Search Filter"></property>
<property name="Identity Strategy">USE_DN</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the ldap-provider remove 2 lines. This is 2 of 2. -->
<!--
Identity Provider for users logging in with username/password against a Kerberos KDC server.
'Default Realm' - Default realm to provide when user enters incomplete user principal (i.e. NIFI.APACHE.ORG).
'Authentication Expiration' - The duration of how long the user authentication is valid for. If the user never logs out, they will be required to log back in following this duration.
-->
<!-- To enable the kerberos-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>kerberos-provider</identifier>
<class>org.apache.nifi.kerberos.KerberosProvider</class>
<property name="Default Realm">NIFI.APACHE.ORG</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the kerberos-provider remove 2 lines. This is 2 of 2. -->
</loginIdentityProviders>

View File

@ -0,0 +1,29 @@
#
# 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.
#
#upgrade test properties
nifi.version=1.1.0
nifi.cluster.is.node=
nifi.cluster.node.address=
nifi.cluster.node.protocol.port=
nifi.cluster.node.protocol.threads=
nifi.cluster.node.event.history.size=
nifi.cluster.node.connection.timeout=
nifi.cluster.node.read.timeout=
nifi.cluster.firewall.file=
nifi.cluster.flow.election.max.wait.time=
nifi.cluster.flow.election.max.candidates=
nifi.cluster.a.new.variable=

View File

@ -0,0 +1,74 @@
#
# 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.
#
# Java command to use when running NiFi
java=java
# Username to use when running NiFi. This value will be ignored on Windows.
run.as=
# Configure where NiFi's lib and conf directories live
lib.dir=./lib
conf.dir=./conf
# How long to wait after telling NiFi to shutdown before explicitly killing the Process
graceful.shutdown.seconds=20
# Disable JSR 199 so that we can use JSP's without running a JDK
java.arg.1=-Dorg.apache.jasper.compiler.disablejsr199=true
# JVM memory settings
java.arg.2=-Xms1024m
java.arg.3=-Xmx1024m
# Enable Remote Debugging
java.arg.debug=-agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=8000
java.arg.4=-Djava.net.preferIPv4Stack=true
# allowRestrictedHeaders is required for Cluster/Node communications to work properly
java.arg.5=-Dsun.net.http.allowRestrictedHeaders=true
java.arg.6=-Djava.protocol.handler.pkgs=sun.net.www.protocol
# The G1GC is still considered experimental but has proven to be very advantageous in providing great
# performance without significant "stop-the-world" delays.
java.arg.13=-XX:+UseG1GC
#Set headless mode by default
java.arg.14=-Djava.awt.headless=true
# Master key in hexadecimal format for encrypted sensitive configuration values
nifi.bootstrap.sensitive.key=
###
# Notification Services for notifying interested parties when NiFi is stopped, started, dies
###
# XML File that contains the definitions of the notification services
notification.services.file=./conf/bootstrap-notification-services.xml
# In the case that we are unable to send a notification for an event, how many times should we retry?
notification.max.attempts=5
# Comma-separated list of identifiers that are present in the notification.services.file; which services should be used to notify when NiFi is started?
#nifi.start.notification.services=email-notification
# Comma-separated list of identifiers that are present in the notification.services.file; which services should be used to notify when NiFi is stopped?
#nifi.stop.notification.services=email-notification
# Comma-separated list of identifiers that are present in the notification.services.file; which services should be used to notify when NiFi dies?
#nifi.dead.notification.services=email-notification

View File

@ -0,0 +1,107 @@
# 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.
# core properties #
nifi.flow.configuration.file=./conf/flow.xml.gz
nifi.flow.configuration.archive.dir=./conf/archive/
nifi.task.configuration.file=./conf/reporting-tasks.xml
nifi.service.configuration.file=./conf/controller-services.xml
nifi.database.directory=./database_repository
nifi.flowfile.repository.directory=./flowfile_repository
nifi.flowfile.repository.partitions=4096
nifi.flowfile.repository.checkpoint.millis=120000
nifi.content.repository.directory.default=./content_repository
nifi.provenance.repository.capacity=25000
nifi.templates.directory=./conf/templates
nifi.version=nifi 0.2.1-SNAPSHOT
nifi.ui.banner.text=DEFAULT BANNER
nifi.ui.autorefresh.interval.seconds=30
nifi.flowcontroller.autoStartProcessors=true
nifi.flowcontroller.schedulestrategy=delay
nifi.flowcontroller.minimum.nanoseconds=1000000
nifi.flowcontroller.graceful.shutdown.seconds=10
nifi.nar.library.directory=./lib
nifi.nar.working.directory=./work/nar/
nifi.flowservice.writedelay.seconds=2
nifi.sensitive.props.key=REPLACE_ME
nifi.sensitive.props.algorithm=PBEWITHMD5AND256BITAES-CBC-OPENSSL
nifi.sensitive.props.provider=BC
nifi.h2.repository.maxmemoryrows=100000
nifi.h2.url.append=;LOCK_TIMEOUT=25000;WRITE_DELAY=0;AUTO_SERVER=FALSE
nifi.h2.max.connections=20
nifi.h2.login.timeout=500
#For testing purposes. Default value should actually be empty!
nifi.remote.input.socket.port=5000
nifi.remote.input.secure=true
# web properties #
nifi.web.war.directory=./lib
nifi.web.http.host=
nifi.web.http.port=
nifi.web.https.host=
nifi.web.https.port=5050
nifi.web.jetty.working.directory=./work/jetty
# security properties #
nifi.security.keystore=target/tmp/keys/localhost/keystore.jks
nifi.security.keystoreType=JKS
nifi.security.keystorePasswd=badKeyPass
nifi.security.keyPasswd=badKeyPass
nifi.security.truststore=target/tmp/keys/localhost/truststore.jks
nifi.security.truststoreType=JKS
nifi.security.truststorePasswd=badTrustPass
nifi.security.needClientAuth=true
nifi.security.user.authorizer=
# cluster common properties (cluster manager and nodes must have same values) #
nifi.cluster.protocol.heartbeat.tick.seconds=10
nifi.cluster.protocol.is.secure=true
nifi.cluster.protocol.socket.timeout.ms=30000
nifi.cluster.protocol.connection.handshake.timeout.seconds=45
# if multicast is used, then nifi.cluster.protocol.multicast.xxx properties must be configured #
nifi.cluster.protocol.use.multicast=false
nifi.cluster.protocol.multicast.address=
nifi.cluster.protocol.multicast.port=
nifi.cluster.protocol.multicast.service.broadcast.delay.ms=500
nifi.cluster.protocol.multicast.service.locator.attempts=3
nifi.cluster.protocol.multicast.service.locator.attempts.delay.seconds=1
#For testing purposes. Default value should actually be empty!
nifi.cluster.remote.input.socket.port=5000
nifi.cluster.remote.input.secure=true
# cluster node properties (only configure for cluster nodes) #
nifi.cluster.is.node=false
nifi.cluster.node.address=
nifi.cluster.node.protocol.port=
nifi.cluster.node.protocol.threads=2
# if multicast is not used, nifi.cluster.node.unicast.xxx must have same values as nifi.cluster.manager.xxx #
nifi.cluster.node.unicast.manager.address=
nifi.cluster.node.unicast.manager.protocol.port=
nifi.cluster.node.unicast.manager.authority.provider.port=
# cluster manager properties (only configure for cluster manager) #
nifi.cluster.is.manager=true
nifi.cluster.manager.address=localhost
nifi.cluster.manager.protocol.port=3030
nifi.cluster.manager.authority.provider.port=4040
nifi.cluster.manager.authority.provider.threads=10
nifi.cluster.manager.node.firewall.file=
nifi.cluster.manager.node.event.history.size=10
nifi.cluster.manager.node.api.connection.timeout.ms=30000
nifi.cluster.manager.node.api.read.timeout.ms=30000
nifi.cluster.manager.node.api.request.threads=10
nifi.cluster.manager.flow.retrieval.delay.seconds=5
nifi.cluster.manager.protocol.threads=10
nifi.cluster.manager.safemode.seconds=0

View File

@ -0,0 +1,204 @@
# 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.
# Core Properties #
nifi.version=1.2.0-SNAPSHOT
nifi.flow.configuration.file=./conf/flow.xml.gz
nifi.flow.configuration.archive.enabled=true
nifi.flow.configuration.archive.dir=./conf/archive/
nifi.flow.configuration.archive.max.time=30 days
nifi.flow.configuration.archive.max.storage=500 MB
nifi.flowcontroller.autoResumeState=true
nifi.flowcontroller.graceful.shutdown.period=10 sec
nifi.flowservice.writedelay.interval=500 ms
nifi.administrative.yield.duration=30 sec
# If a component has no work to do (is "bored"), how long should we wait before checking again for work?
nifi.bored.yield.duration=10 millis
nifi.authorizer.configuration.file=./conf/authorizers.xml
nifi.login.identity.provider.configuration.file=./conf/login-identity-providers.xml
nifi.templates.directory=./conf/templates
nifi.ui.banner.text=
nifi.ui.autorefresh.interval=30 sec
nifi.nar.library.directory=./lib
nifi.nar.working.directory=./work/nar/
nifi.documentation.working.directory=./work/docs/components
####################
# State Management #
####################
nifi.state.management.configuration.file=./conf/state-management.xml
# The ID of the local state provider
nifi.state.management.provider.local=local-provider
# The ID of the cluster-wide state provider. This will be ignored if NiFi is not clustered but must be populated if running in a cluster.
nifi.state.management.provider.cluster=zk-provider
# Specifies whether or not this instance of NiFi should run an embedded ZooKeeper server
nifi.state.management.embedded.zookeeper.start=false
# Properties file that provides the ZooKeeper properties to use if <nifi.state.management.embedded.zookeeper.start> is set to true
nifi.state.management.embedded.zookeeper.properties=./conf/zookeeper.properties
# H2 Settings
nifi.database.directory=./database_repository
nifi.h2.url.append=;LOCK_TIMEOUT=25000;WRITE_DELAY=0;AUTO_SERVER=FALSE
# FlowFile Repository
nifi.flowfile.repository.implementation=org.apache.nifi.controller.repository.WriteAheadFlowFileRepository
nifi.flowfile.repository.directory=./flowfile_repository
nifi.flowfile.repository.partitions=256
nifi.flowfile.repository.checkpoint.interval=2 mins
nifi.flowfile.repository.always.sync=false
nifi.swap.manager.implementation=org.apache.nifi.controller.FileSystemSwapManager
nifi.queue.swap.threshold=20000
nifi.swap.in.period=5 sec
nifi.swap.in.threads=1
nifi.swap.out.period=5 sec
nifi.swap.out.threads=4
# Content Repository
nifi.content.repository.implementation=org.apache.nifi.controller.repository.FileSystemRepository
nifi.content.claim.max.appendable.size=10 MB
nifi.content.claim.max.flow.files=100
nifi.content.repository.directory.default=./content_repository
nifi.content.repository.archive.max.retention.period=12 hours
nifi.content.repository.archive.max.usage.percentage=50%
nifi.content.repository.archive.enabled=true
nifi.content.repository.always.sync=false
nifi.content.viewer.url=/nifi-content-viewer/
# Provenance Repository Properties
nifi.provenance.repository.implementation=org.apache.nifi.provenance.PersistentProvenanceRepository
# Persistent Provenance Repository Properties
nifi.provenance.repository.directory.default=./provenance_repository
nifi.provenance.repository.max.storage.time=24 hours
nifi.provenance.repository.max.storage.size=1 GB
nifi.provenance.repository.rollover.time=30 secs
nifi.provenance.repository.rollover.size=100 MB
nifi.provenance.repository.query.threads=2
nifi.provenance.repository.index.threads=1
nifi.provenance.repository.compress.on.rollover=true
nifi.provenance.repository.always.sync=false
nifi.provenance.repository.journal.count=16
# Comma-separated list of fields. Fields that are not indexed will not be searchable. Valid fields are:
# EventType, FlowFileUUID, Filename, TransitURI, ProcessorID, AlternateIdentifierURI, Relationship, Details
nifi.provenance.repository.indexed.fields=EventType, FlowFileUUID, Filename, ProcessorID, Relationship
# FlowFile Attributes that should be indexed and made searchable. Some examples to consider are filename, uuid, mime.type
nifi.provenance.repository.indexed.attributes=
# Large values for the shard size will result in more Java heap usage when searching the Provenance Repository
# but should provide better performance
nifi.provenance.repository.index.shard.size=500 MB
# Indicates the maximum length that a FlowFile attribute can be when retrieving a Provenance Event from
# the repository. If the length of any attribute exceeds this value, it will be truncated when the event is retrieved.
nifi.provenance.repository.max.attribute.length=65536
# Volatile Provenance Respository Properties
nifi.provenance.repository.buffer.size=100000
# Component Status Repository
nifi.components.status.repository.implementation=org.apache.nifi.controller.status.history.VolatileComponentStatusRepository
nifi.components.status.repository.buffer.size=1440
nifi.components.status.snapshot.frequency=1 min
# Site to Site properties
nifi.remote.input.host=localhost
nifi.remote.input.secure=false
nifi.remote.input.socket.port=8090
nifi.remote.input.http.enabled=true
nifi.remote.input.http.transaction.ttl=30 sec
# web properties #
nifi.web.war.directory=./lib
nifi.web.http.host=
nifi.web.http.port=8080
nifi.web.https.host=
nifi.web.https.port=
nifi.web.jetty.working.directory=./work/jetty
nifi.web.jetty.threads=200
# security properties #
nifi.sensitive.props.key=
nifi.sensitive.props.key.protected=
nifi.sensitive.props.algorithm=PBEWITHMD5AND256BITAES-CBC-OPENSSL
nifi.sensitive.props.provider=BC
nifi.sensitive.props.additional.keys=
nifi.security.keystore=
nifi.security.keystoreType=
nifi.security.keystorePasswd=
nifi.security.keyPasswd=
nifi.security.truststore=
nifi.security.truststoreType=
nifi.security.truststorePasswd=
nifi.security.needClientAuth=
nifi.security.user.authorizer=file-provider
nifi.security.user.login.identity.provider=
nifi.security.ocsp.responder.url=
nifi.security.ocsp.responder.certificate=
# Identity Mapping Properties #
# These properties allow normalizing user identities such that identities coming from different identity providers
# (certificates, LDAP, Kerberos) can be treated the same internally in NiFi. The following example demonstrates normalizing
# DNs from certificates and principals from Kerberos into a common identity string:
#
# nifi.security.identity.mapping.pattern.dn=^CN=(.*?), OU=(.*?), O=(.*?), L=(.*?), ST=(.*?), C=(.*?)$
# nifi.security.identity.mapping.value.dn=$1@$2
# nifi.security.identity.mapping.pattern.kerb=^(.*?)/instance@(.*?)$
# nifi.security.identity.mapping.value.kerb=$1@$2
# cluster common properties (all nodes must have same values) #
nifi.cluster.protocol.heartbeat.interval=5 sec
nifi.cluster.protocol.is.secure=false
# cluster node properties (only configure for cluster nodes) #
nifi.cluster.is.node=true
nifi.cluster.node.address=
nifi.cluster.node.protocol.port=
nifi.cluster.node.protocol.threads=10
nifi.cluster.node.event.history.size=25
nifi.cluster.node.connection.timeout=5 sec
nifi.cluster.node.read.timeout=5 sec
nifi.cluster.firewall.file=
nifi.cluster.flow.election.max.wait.time=5 mins
nifi.cluster.flow.election.max.candidates=
# zookeeper properties, used for cluster management #
nifi.zookeeper.connect.string=
nifi.zookeeper.connect.timeout=3 secs
nifi.zookeeper.session.timeout=3 secs
nifi.zookeeper.root.node=/nifi
# kerberos #
nifi.kerberos.krb5.file=
# kerberos service principal #
nifi.kerberos.service.principal=
nifi.kerberos.service.keytab.location=
# kerberos spnego principal #
nifi.kerberos.spnego.principal=
nifi.kerberos.spnego.keytab.location=
nifi.kerberos.spnego.authentication.expiration=12 hours
# external properties files for variable registry
# supports a comma delimited list of file locations
nifi.variable.registry.properties=
# Build info
nifi.build.tag=HEAD
nifi.build.branch=master
nifi.build.revision=868795c
nifi.build.timestamp=2016-12-13T15:41:36Z

View File

@ -0,0 +1,41 @@
#
# 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.
#
# This properties file specifies how to update the provided nifi.properties
# Comma separated list of properties to put the hostname into
hostname.properties= \
nifi.remote.input.host, \
nifi.web.https.host, \
nifi.cluster.node.address
# Comma separated list of properties to increment (must also be defined in this file)
incrementing.properties= \
nifi.web.https.port, \
nifi.remote.input.socket.port, \
nifi.cluster.node.protocol.port
nifi.web.https.port=9443
nifi.remote.input.socket.port=10443
nifi.cluster.node.protocol.port=11443
# Properties to set verbatim
nifi.remote.input.secure=true
nifi.cluster.protocol.is.secure=true
nifi.web.http.host=
nifi.web.http.port=

View File

@ -0,0 +1,21 @@
#
# 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.
#
# JVM memory settings
java.arg.2=-Xms512m
java.arg.3=-Xmx512m

View File

@ -0,0 +1,112 @@
<?xml version="1.0" encoding="UTF-8" standalone="yes"?>
<!--
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.
-->
<!--
This file lists the login identity providers to use when running securely. In order
to use a specific provider it must be configured here and it's identifier
must be specified in the nifi.properties file.
-->
<loginIdentityProviders>
<!--
Identity Provider for users logging in with username/password against an LDAP server.
'Authentication Strategy' - How the connection to the LDAP server is authenticated. Possible
values are ANONYMOUS, SIMPLE, LDAPS, or START_TLS.
'Manager DN' - The DN of the manager that is used to bind to the LDAP server to search for users.
'Manager Password' - The password of the manager that is used to bind to the LDAP server to
search for users.
'TLS - Keystore' - Path to the Keystore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Keystore Password' - Password for the Keystore that is used when connecting to LDAP
using LDAPS or START_TLS.
'TLS - Keystore Type' - Type of the Keystore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Truststore' - Path to the Truststore that is used when connecting to LDAP using LDAPS or START_TLS.
'TLS - Truststore Password' - Password for the Truststore that is used when connecting to
LDAP using LDAPS or START_TLS.
'TLS - Truststore Type' - Type of the Truststore that is used when connecting to LDAP using
LDAPS or START_TLS (i.e. JKS or PKCS12).
'TLS - Client Auth' - Client authentication policy when connecting to LDAP using LDAPS or START_TLS.
Possible values are REQUIRED, WANT, NONE.
'TLS - Protocol' - Protocol to use when connecting to LDAP using LDAPS or START_TLS. (i.e. TLS,
TLSv1.1, TLSv1.2, etc).
'TLS - Shutdown Gracefully' - Specifies whether the TLS should be shut down gracefully
before the target context is closed. Defaults to false.
'Referral Strategy' - Strategy for handling referrals. Possible values are FOLLOW, IGNORE, THROW.
'Connect Timeout' - Duration of connect timeout. (i.e. 10 secs).
'Read Timeout' - Duration of read timeout. (i.e. 10 secs).
'Url' - Space-separated list of URLs of the LDAP servers (i.e. ldap://<hostname>:<port>).
'User Search Base' - Base DN for searching for users (i.e. CN=Users,DC=example,DC=com).
'User Search Filter' - Filter for searching for users against the 'User Search Base'.
(i.e. sAMAccountName={0}). The user specified name is inserted into '{0}'.
'Identity Strategy' - Strategy to identify users. Possible values are USE_DN and USE_USERNAME.
The default functionality if this property is missing is USE_DN in order to retain
backward compatibility. USE_DN will use the full DN of the user entry if possible.
USE_USERNAME will use the username the user logged in with.
'Authentication Expiration' - The duration of how long the user authentication is valid
for. If the user never logs out, they will be required to log back in following
this duration.
-->
<!-- To enable the ldap-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>ldap-provider</identifier>
<class>org.apache.nifi.ldap.LdapProvider</class>
<property name="Authentication Strategy">START_TLS</property>
<property name="Manager DN"></property>
<property name="Manager Password"></property>
<property name="TLS - Keystore"></property>
<property name="TLS - Keystore Password"></property>
<property name="TLS - Keystore Type"></property>
<property name="TLS - Truststore"></property>
<property name="TLS - Truststore Password"></property>
<property name="TLS - Truststore Type"></property>
<property name="TLS - Client Auth"></property>
<property name="TLS - Protocol"></property>
<property name="TLS - Shutdown Gracefully"></property>
<property name="Referral Strategy">FOLLOW</property>
<property name="Connect Timeout">10 secs</property>
<property name="Read Timeout">10 secs</property>
<property name="Url"></property>
<property name="User Search Base"></property>
<property name="User Search Filter"></property>
<property name="Identity Strategy">USE_DN</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the ldap-provider remove 2 lines. This is 2 of 2. -->
<!--
Identity Provider for users logging in with username/password against a Kerberos KDC server.
'Default Realm' - Default realm to provide when user enters incomplete user principal (i.e. NIFI.APACHE.ORG).
'Authentication Expiration' - The duration of how long the user authentication is valid for. If the user never logs out, they will be required to log back in following this duration.
-->
<!-- To enable the kerberos-provider remove 2 lines. This is 1 of 2.
<provider>
<identifier>kerberos-provider</identifier>
<class>org.apache.nifi.kerberos.KerberosProvider</class>
<property name="Default Realm">NIFI.APACHE.ORG</property>
<property name="Authentication Expiration">12 hours</property>
</provider>
To enable the kerberos-provider remove 2 lines. This is 2 of 2. -->
</loginIdentityProviders>

View File

@ -0,0 +1,28 @@
#
# 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.
#
#upgrade test properties
nifi.cluster.is.node=
nifi.cluster.node.address=
nifi.cluster.node.protocol.port=
nifi.cluster.node.protocol.threads=
nifi.cluster.node.event.history.size=
nifi.cluster.node.connection.timeout=
nifi.cluster.node.read.timeout=
nifi.cluster.firewall.file=
nifi.cluster.flow.election.max.wait.time=
nifi.cluster.flow.election.max.candidates=
nifi.cluster.a.new.variable=

View File

@ -72,6 +72,10 @@ language governing permissions and limitations under the License. -->
<groupId>org.apache.nifi</groupId> <groupId>org.apache.nifi</groupId>
<artifactId>nifi-toolkit-s2s</artifactId> <artifactId>nifi-toolkit-s2s</artifactId>
</dependency> </dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-toolkit-admin</artifactId>
</dependency>
<dependency> <dependency>
<groupId>org.apache.nifi</groupId> <groupId>org.apache.nifi</groupId>
<artifactId>nifi-toolkit-zookeeper-migrator</artifactId> <artifactId>nifi-toolkit-zookeeper-migrator</artifactId>

View File

@ -0,0 +1,39 @@
@echo off
rem
rem Licensed to the Apache Software Foundation (ASF) under one or more
rem contributor license agreements. See the NOTICE file distributed with
rem this work for additional information regarding copyright ownership.
rem The ASF licenses this file to You under the Apache License, Version 2.0
rem (the "License"); you may not use this file except in compliance with
rem the License. You may obtain a copy of the License at
rem
rem http://www.apache.org/licenses/LICENSE-2.0
rem
rem Unless required by applicable law or agreed to in writing, software
rem distributed under the License is distributed on an "AS IS" BASIS,
rem WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
rem See the License for the specific language governing permissions and
rem limitations under the License.
rem
rem Use JAVA_HOME if it's set; otherwise, just use java
if "%JAVA_HOME%" == "" goto noJavaHome
if not exist "%JAVA_HOME%\bin\java.exe" goto noJavaHome
set JAVA_EXE=%JAVA_HOME%\bin\java.exe
goto startConfig
:noJavaHome
echo The JAVA_HOME environment variable is not defined correctly.
echo Instead the PATH will be used to find the java executable.
echo.
set JAVA_EXE=java
goto startConfig
:startConfig
set LIB_DIR=%~sdp0..\classpath;%~sdp0..\lib
SET JAVA_PARAMS=-cp %LIB_DIR%\* -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.toolkit.admin.nodemanager.NodeManagerTool
cmd.exe /C ""%JAVA_EXE%" %JAVA_PARAMS% %* ""

View File

@ -0,0 +1,119 @@
#!/bin/sh
#
# 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.
#
#
# Script structure inspired from Apache Karaf and other Apache projects with similar startup approaches
SCRIPT_DIR=$(dirname "$0")
SCRIPT_NAME=$(basename "$0")
NIFI_TOOLKIT_HOME=$(cd "${SCRIPT_DIR}" && cd .. && pwd)
PROGNAME=$(basename "$0")
warn() {
(>&2 echo "${PROGNAME}: $*")
}
die() {
warn "$*"
exit 1
}
detectOS() {
# OS specific support (must be 'true' or 'false').
cygwin=false;
aix=false;
os400=false;
darwin=false;
case "$(uname)" in
CYGWIN*)
cygwin=true
;;
AIX*)
aix=true
;;
OS400*)
os400=true
;;
Darwin)
darwin=true
;;
esac
# For AIX, set an environment variable
if ${aix}; then
export LDR_CNTRL=MAXDATA=0xB0000000@DSA
echo ${LDR_CNTRL}
fi
}
locateJava() {
# Setup the Java Virtual Machine
if $cygwin ; then
[ -n "${JAVA}" ] && JAVA=$(cygpath --unix "${JAVA}")
[ -n "${JAVA_HOME}" ] && JAVA_HOME=$(cygpath --unix "${JAVA_HOME}")
fi
if [ "x${JAVA}" = "x" ] && [ -r /etc/gentoo-release ] ; then
JAVA_HOME=$(java-config --jre-home)
fi
if [ "x${JAVA}" = "x" ]; then
if [ "x${JAVA_HOME}" != "x" ]; then
if [ ! -d "${JAVA_HOME}" ]; then
die "JAVA_HOME is not valid: ${JAVA_HOME}"
fi
JAVA="${JAVA_HOME}/bin/java"
else
warn "JAVA_HOME not set; results may vary"
JAVA=$(type java)
JAVA=$(expr "${JAVA}" : '.* \(/.*\)$')
if [ "x${JAVA}" = "x" ]; then
die "java command not found"
fi
fi
fi
}
init() {
# Determine if there is special OS handling we must perform
detectOS
# Locate the Java VM to execute
locateJava "$1"
}
run() {
LIBS="${NIFI_TOOLKIT_HOME}/lib/*"
sudo_cmd_prefix=""
if $cygwin; then
NIFI_TOOLKIT_HOME=$(cygpath --path --windows "${NIFI_TOOLKIT_HOME}")
CLASSPATH="$(cygpath --path --windows "${LIBS}")"
else
CLASSPATH="${LIBS}"
fi
export JAVA_HOME="$JAVA_HOME"
export NIFI_TOOLKIT_HOME="$NIFI_TOOLKIT_HOME"
umask 0077
"${JAVA}" -cp "${CLASSPATH}" -Xms12m -Xmx24m org.apache.nifi.toolkit.admin.nodemanager.NodeManagerTool "$@"
return $?
}
init "$1"
run "$@"

View File

@ -0,0 +1,39 @@
@echo off
rem
rem Licensed to the Apache Software Foundation (ASF) under one or more
rem contributor license agreements. See the NOTICE file distributed with
rem this work for additional information regarding copyright ownership.
rem The ASF licenses this file to You under the Apache License, Version 2.0
rem (the "License"); you may not use this file except in compliance with
rem the License. You may obtain a copy of the License at
rem
rem http://www.apache.org/licenses/LICENSE-2.0
rem
rem Unless required by applicable law or agreed to in writing, software
rem distributed under the License is distributed on an "AS IS" BASIS,
rem WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
rem See the License for the specific language governing permissions and
rem limitations under the License.
rem
rem Use JAVA_HOME if it's set; otherwise, just use java
if "%JAVA_HOME%" == "" goto noJavaHome
if not exist "%JAVA_HOME%\bin\java.exe" goto noJavaHome
set JAVA_EXE=%JAVA_HOME%\bin\java.exe
goto startConfig
:noJavaHome
echo The JAVA_HOME environment variable is not defined correctly.
echo Instead the PATH will be used to find the java executable.
echo.
set JAVA_EXE=java
goto startConfig
:startConfig
set LIB_DIR=%~sdp0..\classpath;%~sdp0..\lib
SET JAVA_PARAMS=-cp %LIB_DIR%\* -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.toolkit.admin.notify.NotificationTool
cmd.exe /C ""%JAVA_EXE%" %JAVA_PARAMS% %* ""

View File

@ -0,0 +1,120 @@
#!/bin/sh
#
# 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.
#
#
# Script structure inspired from Apache Karaf and other Apache projects with similar startup approaches
SCRIPT_DIR=$(dirname "$0")
SCRIPT_NAME=$(basename "$0")
NIFI_TOOLKIT_HOME=$(cd "${SCRIPT_DIR}" && cd .. && pwd)
PROGNAME=$(basename "$0")
warn() {
(>&2 echo "${PROGNAME}: $*")
}
die() {
warn "$*"
exit 1
}
detectOS() {
# OS specific support (must be 'true' or 'false').
cygwin=false;
aix=false;
os400=false;
darwin=false;
case "$(uname)" in
CYGWIN*)
cygwin=true
;;
AIX*)
aix=true
;;
OS400*)
os400=true
;;
Darwin)
darwin=true
;;
esac
# For AIX, set an environment variable
if ${aix}; then
export LDR_CNTRL=MAXDATA=0xB0000000@DSA
echo ${LDR_CNTRL}
fi
}
locateJava() {
# Setup the Java Virtual Machine
if $cygwin ; then
[ -n "${JAVA}" ] && JAVA=$(cygpath --unix "${JAVA}")
[ -n "${JAVA_HOME}" ] && JAVA_HOME=$(cygpath --unix "${JAVA_HOME}")
fi
if [ "x${JAVA}" = "x" ] && [ -r /etc/gentoo-release ] ; then
JAVA_HOME=$(java-config --jre-home)
fi
if [ "x${JAVA}" = "x" ]; then
if [ "x${JAVA_HOME}" != "x" ]; then
if [ ! -d "${JAVA_HOME}" ]; then
die "JAVA_HOME is not valid: ${JAVA_HOME}"
fi
JAVA="${JAVA_HOME}/bin/java"
else
warn "JAVA_HOME not set; results may vary"
JAVA=$(type java)
JAVA=$(expr "${JAVA}" : '.* \(/.*\)$')
if [ "x${JAVA}" = "x" ]; then
die "java command not found"
fi
fi
fi
}
init() {
# Determine if there is special OS handling we must perform
detectOS
# Locate the Java VM to execute
locateJava "$1"
}
run() {
LIBS="${NIFI_TOOLKIT_HOME}/lib/*"
sudo_cmd_prefix=""
if $cygwin; then
NIFI_TOOLKIT_HOME=$(cygpath --path --windows "${NIFI_TOOLKIT_HOME}")
CLASSPATH="$(cygpath --path --windows "${LIBS}")"
else
CLASSPATH="${LIBS}"
fi
export JAVA_HOME="$JAVA_HOME"
export NIFI_TOOLKIT_HOME="$NIFI_TOOLKIT_HOME"
umask 0077
"${JAVA}" -cp "${CLASSPATH}" -Xms12m -Xmx24m org.apache.nifi.toolkit.admin.notify.NotificationTool "$@"
return $?
}
init "$1"
run "$@"

View File

@ -20,12 +20,16 @@
<artifactId>nifi</artifactId> <artifactId>nifi</artifactId>
<version>1.2.0-SNAPSHOT</version> <version>1.2.0-SNAPSHOT</version>
</parent> </parent>
<properties>
<client.version>1.2.0-SNAPSHOT</client.version>
</properties>
<artifactId>nifi-toolkit</artifactId> <artifactId>nifi-toolkit</artifactId>
<packaging>pom</packaging> <packaging>pom</packaging>
<modules> <modules>
<module>nifi-toolkit-tls</module> <module>nifi-toolkit-tls</module>
<module>nifi-toolkit-encrypt-config</module> <module>nifi-toolkit-encrypt-config</module>
<module>nifi-toolkit-s2s</module> <module>nifi-toolkit-s2s</module>
<module>nifi-toolkit-admin</module>
<module>nifi-toolkit-zookeeper-migrator</module> <module>nifi-toolkit-zookeeper-migrator</module>
<module>nifi-toolkit-flowfile-repo</module> <module>nifi-toolkit-flowfile-repo</module>
<module>nifi-toolkit-assembly</module> <module>nifi-toolkit-assembly</module>

View File

@ -936,6 +936,11 @@
<artifactId>nifi-toolkit-zookeeper-migrator</artifactId> <artifactId>nifi-toolkit-zookeeper-migrator</artifactId>
<version>1.2.0-SNAPSHOT</version> <version>1.2.0-SNAPSHOT</version>
</dependency> </dependency>
<dependency>
<groupId>org.apache.nifi</groupId>
<artifactId>nifi-toolkit-admin</artifactId>
<version>1.2.0-SNAPSHOT</version>
</dependency>
<dependency> <dependency>
<groupId>org.apache.nifi</groupId> <groupId>org.apache.nifi</groupId>
<artifactId>nifi-registry-service</artifactId> <artifactId>nifi-registry-service</artifactId>