Commit Graph

1597 Commits

Author SHA1 Message Date
Mark Payne ba83bf9c05 NIFI-631: Initial implementation of FetchFile processor
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-25 18:03:19 -05:00
Joe Skora 4c4d62c61f NIFI-631: Added ListFile processor.
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-25 18:03:19 -05:00
Mark Payne 5061e5fa0a NIFI-1222: Session.adjustCounter keeps track of local and global counters; it then call processContext.adjustCounter with each of them, but ProcessContext was changed a while back to automatically increment both 'lcoal' and 'global' counters each time, so our numbers are doubled; removed the 'localCounters' and 'globalCounters' from StandardProcessSession and replaced with just 'counters'
Signed-off-by: joewitt <joewitt@apache.org>
2015-11-25 16:26:59 -05:00
Mark Payne a29b7b3bf0 NIFI-1203: Do not count looping connections when determining validity based on incoming connections
Signed-off-by: joewitt <joewitt@apache.org>
2015-11-25 15:32:41 -05:00
Matt Gilman 0435911186 NIFI-1198: - Updating the connection source and destination when appropriate (deletion and (re)connection).
Signed-off-by: joewitt <joewitt@apache.org>
2015-11-25 14:50:56 -05:00
Mark Payne e1742c5a04 Merge branch 'master' of https://git-wip-us.apache.org/repos/asf/nifi 2015-11-25 13:03:22 -05:00
Mark Payne 4fa2a713a2 NIFI-1192: Removed some additional white space 2015-11-25 12:57:06 -05:00
Mark Payne 657885e5ba Merge branch 'NIFI-1192B' of https://github.com/olegz/nifi into NIFI-1192 2015-11-25 12:30:15 -05:00
Oleg Zhurakousky d949ee1a1e NIFI-1192 added support for dynamic properties to GetKafka
Due to the fact that current component uses artificial names for properties set via UI and then maps those properties to the actual names used by Kafka, we can not rely on NiFi UI to display an error if user attempts to set a dynamic property which will eventually map to the same Kafka property. So, I’ve decided that any dynamic property will simply override an existing property with WARNING message displayed. It is actually consistent with how Kafka does it and displayed the overrides in the console. Updated the relevant annotation description.
It is also worth to mentioned that current code was using an old property from Kafka 0.7 (“zk.connectiontimeout.ms”) which is no longer present in Kafka 0.8 (WARN Timer-Driven Process Thread-7 utils.VerifiableProperties:83 - Property zk.connectiontimeout.ms is not valid). The add/override strategy would provide for more flexibility when dealing with Kafka volatile configuration until things will settle down and we can get some sensible defaults in place.

While doing it addressed the following issues that were discovered while making modification and testing:
ISSUE: When GetKafka started and there are no messages in Kafka topic the onTrigger(..) method would block due to the fact that Kafka’s ConsumerIterator.hasNext() blocks. When attempt was made to stop GetKafka would stops successfully due to the interrupt. However in UI it would appear as ERROR based on the fact that InterruptException was not handled.
RESOLUTION: After discussing it with @markap14 the the general desire is to let the task exit as quick as possible and that the whole thread maintenance logic was there initially due to the fact that there was no way to tell Kafka consumer to return immediately if there are no events. In this patch we are now using ‘consumer.timeout.ms’ property of Kafka and setting its value to 1 millisecond (default is -1 - always block infinitely). This ensures that tasks that attempted to read an empty topic will exit immediately just to be rescheduled by NiFi based on user configurations.

ISSUE:  Kafka would not release FlowFile with events if it didn’t have enough to complete the batch since it would block waiting for more messages (based on the blocking issue described above).
RESOLUTION: The invocation of hasNext() results in Kafka’s ConsumerTimeoutException which is handled in the catch block where the FlowFile with partial batch will be released to success. Not sure if we need to put a WARN message. In fact in my opinion we should not as it may create unnecessary confusion.

ISSUE: When configuring a consumer for topic and specifying multiple concurrent consumers in ‘topicCountMap’ based on 'context.getMaxConcurrentTasks()’ each consumer would bind to a topic partition. If you have less partitions then the value returned by 'context.getMaxConcurrentTasks()’ you would essentially allocate Kafka resources that would never get a chance to receive a single message  (see more here https://cwiki.apache.org/confluence/display/KAFKA/Consumer+Group+Example).
RESOLUTION: Logic was added to determine the amount of partitions for a topic and in the event where 'context.getMaxConcurrentTasks()’ value is greater than the amount of partitions, the partition count will be used to when creating ‘topicCountMap’ and WARNING message will be displayed)see code). Unfortunately we can’t do anything with the actual tasks, but based on current state of the code they will exit immediately just to be rescheduled where the process will repeat. NOTE: That is not ideal as it will be rescheduling tasks that will never have a chance to do anything, but at least it could be fixed on the user side after reading the warning message.

NIFI-1192 added dynamic properties support for PutKafka

NIFI-1192 polishing

NIFI-1192 polished and addressed PR comments
2015-11-24 12:14:36 -05:00
Bryan Bende 4281a51c83 Merge branch 'NIFI-1208' 2015-11-23 21:35:57 -05:00
Mark Payne 4e2c94d659 Merge branch 'master' of https://git-wip-us.apache.org/repos/asf/nifi 2015-11-23 14:18:50 -05:00
Mark Payne 2516b1dad2 NIFI-1171: Ensure that we pick up changes when files roll over and ensure that we don't pick up the rolled over file multiple times 2015-11-23 14:11:14 -05:00
Bryan Bende 3ffb455903 NIFI-1208 Adding context.yield() to Listen and Put syslog when there is nothing to do in onTrigger() 2015-11-23 10:30:33 -05:00
Jenn Barnabee b74617ba6a NIFI-1210: Fixes Copy and Paste descriptions and changes several references to canvas instead of graph to maintain consistency. Also capitalizes Site-to-Site in a couple of places for consistency.
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-22 08:29:14 -05:00
Jenn Barnabee 327342916e NIFI-1210: Makes various changes to the User Guide for the 0.4.0 version release.
Reviewed and amended (see ticket for details) by Tony Kurc (tkurc@apache.org)
2015-11-21 22:57:19 -05:00
Aldrin Piri 1580edb558 NIFI-1196 Correcting treatment of FETCH events as a continuation of a lineage trail and not the start of a new one.
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-21 20:21:36 -05:00
Joseph Percivall cdd2c4f22c NIFI-1086: Changed behavior on retrieval with no input file to RETRIEVE events, removed @TriggerWhenEmpty
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-21 19:20:33 -05:00
Bryan Bende f1f67f6395 Fixing one-character typo in syslog attribute name 2015-11-20 10:45:21 -05:00
Aldrin Piri 08d59e4374 NIFI-1196 Providing handling of FETCH provenance events for their "unique" property, transit URI, within the framework and UI.
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-19 17:42:15 -05:00
Bryan Bende 40dd8a0a84 NIFI-1174 Refactoring the HBase client API and adding a PutHBaseJSON which can write a whole row from a single json document - Adding Complex Field Strategy to PutHBaseJSON to allow more control of complex fields - Improving error messages to indicate what the problem was with an invalid row
Signed-off-by: Bryan Bende <bbende@apache.org>
2015-11-19 13:49:02 -05:00
Joseph Percivall 8c2323dc8d NIFI-1086 Provide refactoring of InvokeHTTP
NIFI-980 Add support for HTTP Digest authentication to InvokeHttp
NIFI-1080 Provide additional InvokeHttp unit tests
NIFI-1133 InvokeHTTP Processor does not save Location header for 3xx responses
NIFI-1009 InvokeHTTP should be able to be scheduled without any incoming connection for GET operations
NIFI-61 Multiple improvements for InvokeHTTP inclusive of providing unique tx.id across clusters, dynamic HTTP header properties

Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-19 01:40:21 -05:00
Joe Percivall fb335ea282 NIFI-1165: Fix for tests TestRouteText and PutHDFS which did not succeed on Windows
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-19 01:01:34 -05:00
Mark Payne e862f7ff03 NIFI-1165: Use FileChannel instead of RandomAccessFile in order to avoid locking files in Windows
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-19 01:01:28 -05:00
Joseph Percivall 1e5cc070a3 NIFI-1081 Adding option to ExecuteStreamCommand to put output value to an attribute
Reviewed and amended (comments,whitespace,and some code readability (discussed in ticket)) by Tony Kurc (tkurc@apache.org)
2015-11-18 23:23:10 -05:00
Tony Kurc 9e2f6df205 NIFI-1123: Fixing a botched commit. 2015-11-18 21:59:42 -05:00
Joe Skora 52b24b93d9 NIFI-1123 Adds expression language support to DeleteAttributesExpression on UpdateAttributes Processor.
Reviewed by Tony Kurc (trkurc@gmail.com) after Aldrin Piri <aldrin@apache.org> did the initial review and actionable comments
2015-11-18 19:10:21 -05:00
Tony Kurc ab7940368a NIFI-1187: Fixing issue of possible assigment reordering causing uninitalized values to be possibly returned
Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-18 17:37:47 -05:00
Aldrin Piri c541c82c35 NIFI-1191 Adding missing tags for ConvertAvroToJSON 2015-11-18 16:38:15 -05:00
Mark Payne 69bce2c2db NIFI-1168: Ensure that processors with only looping
connections are scheduled to run, even if the connections have no FlowFiles;
 expose these details to processor developers; update documentation

Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-18 14:53:30 -05:00
Aldrin Piri 773576e041 NIFI-1108 Providing additional annotations on processors to ensure utilization of the InputRequirement annotation. 2015-11-18 13:56:21 -05:00
Mark Payne 911e1c5412 NIFI-1108: Updated processors to include the @InputRequirement annotation
Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-18 13:47:59 -05:00
Mark Payne d88b6cb6bc NIFI-1173: Even if FlowFile Queue is empty, it needs to hold onto the Empty Queue request so that subsequent cancel/clear requests can reference it
Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-18 10:56:38 -05:00
Mark Payne 180ea1ba22 NIFI-1176: Use a smaller internal blocking queue size of only 5000 messages; do not create a new queue every time the processor is scheduled to run
Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-18 10:51:22 -05:00
Mark Payne 93be753301 NIFI-1181: Ensure that a FlowFile's uuid cannot be modified by processors
Signed-off-by: Aldrin Piri <aldrin@apache.org>
2015-11-18 10:35:37 -05:00
Bryan Bende 2b9b5e008f NIFI-1175 Exposing minimum properties required to create an HBase connection on the HBaseClientService as an optional alternative to the conf files 2015-11-17 12:01:46 -05:00
Mark Payne 453b140d6b Merge branch 'NIFI-748' of https://github.com/olegz/nifi into NIFI-748 2015-11-17 09:23:10 -05:00
Mark Payne d16392c618 NIFI-1114: Added documentation for nifi.kerberos.krb5.file property 2015-11-17 08:45:28 -05:00
Jenn Barnabee f98e335cde NIFI-1178: Updates to Administration Guide - System Properties section.
Signed-off-by: Mark Payne <markap14@hotmail.com>
2015-11-17 08:16:45 -05:00
Naveen Madhire 03a54bf2d5 Changes after review 2015-11-16 13:32:17 -06:00
Naveen Madhire b954ca620e NIFI-1146 Allow GetKafka to be configured with auto.offset.reset to largest or smallest 2015-11-16 11:59:52 -06:00
Oleg Zhurakousky 15880f9fcc NIFI-748 addressed PR comments
- made DocReader package private
- polished logic in read(..) method to avoid escaping the loop
- added call to sorting logic in LuceneUtil.groupDocsByStorageFileName(..) to ensure that previous behavior and assumptions in read(..) methodd are preserved
- other minor polishing
2015-11-16 08:39:23 -05:00
Matt Gilman 64369f67f9 NIFI-1160:
- Addressing possible NPE when removing a drop request.
- Only updating a connection status once it's been retrieved for the first time.
2015-11-16 08:34:04 -05:00
Mark Payne da9f40b3f0 NIFI-973: Updated main Help page to provide a link to the Getting Started Guide.
Reviewed by Aldrin Piri (aldrin@apache.org)
2015-11-15 08:38:51 -05:00
Tony Kurc 528dab78d6 NIFI-1073: Fixing coverity discovered errors. Resource leaks, and statics
Reviewed by Bryan Bende (bbende@apache.org)
2015-11-14 18:56:43 -05:00
Aldrin Piri 01539ed323 NIFI-1163: Providing handling of SSLContext creation in GetHTTP in case of only performing a one-way SSL request and accompanying test to verify the configuration/usage.
Reviewed by Tony Kurc (tkurc@apache.org)
2015-11-14 18:47:50 -05:00
Oleg Zhurakousky a4d93c62c8 NIFI-748 Fixed logic around handling partial query results from provenance repository
- Ensured that failures derived form correlating Document to its actual provenance event do fail the entire query and produce partial results with warning messages
- Refactored DocsReader.read() operation.
- Added test to validate two conditions where the such failures could occur
2015-11-13 14:23:31 -05:00
Mark Payne 90f6830003 Merge branch 'master' of https://git-wip-us.apache.org/repos/asf/nifi 2015-11-13 10:47:05 -05:00
Mark Payne 22de23baa6 NIFI-1097: Rewrite PutKafka to use the new producer api 2015-11-13 10:46:46 -05:00
joewitt 36d00a60f5 NIFI-1155 fixed contrib-check violation 2015-11-13 00:06:12 -05:00
Mark Payne 37d6b7350e NIFI-1155: Refactored StandardFlowFileQueue to update member variables more intelligently, using CAS operations instead of locks. This reduces code complexities because other optimizations that previously existed are no longer needed
Signed-off-by: joewitt <joewitt@apache.org>
2015-11-13 00:06:12 -05:00