MemQ: An Efficient, Scalable Cloud Native PubSub System

1,506
Pinterest
Pinterest's profile on StackShare is not actively maintained, so the information here may be out of date.

By Ambud Sharma | Tech Lead and Engineering Manager, Logging Platform


The Logging Platform powers all data ingestion and transportation at Pinterest. At the heart of the Pinterest Logging Platform are Distributed PubSub systems that help our customers transport / buffer data and consume asynchronously.

In this blog we introduce MemQ (pronounced mem — queue), an efficient, scalable PubSub system developed for the cloud at Pinterest that has been powering Near Real-Time data transportation use cases for us since mid-2020 and complements Kafka while being up to 90% more cost efficient.

History

For nearly a decade, Pinterest has relied on Apache Kafka as the sole PubSub system. As Pinterest grew, so did the amount of data and the challenges around operating a very large scale distributed PubSub platform. Operating Apache Kafka at Scale gave us a great deal of insight on how to build a scalable PubSub system. Upon deep investigation of the operational and scalability challenges of our PubSub environment, we arrived at the following key takeaways:

  1. Not every dataset needs sub-second latency service, latency and cost should be inversely proportional (lower latency should cost more)
  2. Storage and Serving components of a PubSub system need to be separated to enable independent scalability based on resources.
  3. Ordering on read instead of write provides required flexibility for specific consumer use cases (different applications can have different for same dataset)
  4. Strict partition ordering is not necessary at Pinterest in most cases and often leads to scalability challenges.
  5. Rebalancing in Kafka is expensive, often results in performance degradation, and has a negative impact for customers on a saturated cluster.
  6. Running custom replication in a cloud environment is expensive.

In 2018, we experimented with a new type of PubSub system that would natively leverage the cloud. In 2019, we started formally exploring options on how to solve our PubSub scalability challenges and evaluated multiple PubSub technologies based on cost of operations as well as reengineering cost for existing technologies to meet the demands of Pinterest. We finally landed at the conclusion that we needed a PubSub technology that built on the learnings of Apache Kafka, Apache Pulsar, and Facebook LogDevice, and was built for the cloud.

MemQ is a new PubSub system that augments Kafka at Pinterest. It uses a decoupled storage and serving architecture similar to Apache Pulsar and Facebook Logdevice; however, it relies on a pluggable replicated storage layer i.e. Object Store / DFS / NFS for storing data. The net result is a PubSub system that:

  • Handles GB/s traffic
  • Independently scales, writes, and reads
  • Doesn’t require expensive rebalancing to handle traffic growth
  • Is 90% more cost effective than our Kafka footprint

Secret Sauce

The secret of MemQ is that it leverages micro-batching and immutable writes to create an architecture where the number of Input/output Operations Per Second (IOPS) necessary on the storage layer are dramatically reduced, allowing the cost effective use of a cloud native Object Store like Amazon S3. This approach is analogous to packet switching for networks (vs circuit switching, i.e. single large continuous storage of data such as kafka partition).

MemQ breaks the continuous stream of logs into blocks (objects), similar to ledgers in Pulsar but different in that they are written as objects and are immutable. The size of these “packets” / “objects,” known internally in MemQ as a Batch, play a role in determining the End-to-End (E2E) latency. The smaller the packets, the faster they can be written at the cost of more IOPS. MemQ therefore allows tunable E2E latency at the cost of higher IOPs. A key performance benefit of this architecture is enabling separation of read and write hardware dependending on the underlying storage layer, allowing writes and reads to scale independently as packets that can be spread across the storage layer.

This also eliminated the constraints experienced in Kafka where in order to recover a replica, a partition must be re-replicated from the beginning. In the case of MemQ, the underlying replicated storage only needs to recover the specific Batch whose replica counts were reduced due to faults in case of storage failures. However, since MemQ at Pinterest runs on Amazon S3, the recovery, sharding, and scaling of storage is handled by AWS without any manual intervention from Pinterest.

Components of MemQ

Client

MemQ client discovers the cluster using a seed node and then connects to the seed node to discover metadata and the Brokers hosting the TopicProcessors for a given Topic or, in case of the consumer, the address of the notification queue.

Broker

Similar to other PubSub systems, MemQ has the concept of a Broker. A MemQ Broker is a part of the cluster and is primarily responsible for handling metadata and write requests.

Note: read requests in MemQ can be handled directly by the Storage layer unless the read Brokers are used

Cluster Governor

The Governor is a leader in the MemQ cluster and is responsible for automated rebalancing and TopicProcessor assignments. Any Broker in the cluster can be elected a Governor, and it communicates with Brokers using Zookeeper, which is also used for Governor election.

The Governor makes assignment decisions using a pluggable assignment algorithm. The default one evaluates available capacity on a Broker to make allocation decisions. Governor also uses this capability to handle Broker failures and restore capacity for topics.

Topic & TopicProcessor

MemQ, similar to other PubSub systems, uses the logical concept of Topic. MemQ topics on a Broker are handled by a module called TopicProcessor. A Broker can host one or more TopicProcessors, where each TopicProcessor instance handles one topic. Topics have write and read partitions. The write partitions are used to create TopicProcessors (1:1 relation), and the read partitions are used to determine the level of parallelism needed by the consumer to process the data. The read partition count is equal to the number of partitions of the notification queue.

Storage

MemQ storage is made of two parts:

  1. Replicated Storage (Object Store / DFS)
  2. Notification Queue (Kafka, Pulsar, etc.)

1. Replicated Storage

MemQ allows for pluggable storage handlers. At present, we have implemented a storage handler for Amazon S3. Amazon S3 offers a cost effective solution for fault-tolerant, on-demand storage. The following prefix format on S3 is used by MemQ to create the high throughput and scalable storage layer:

s3://<bucketname>/<(a) 2 byte hash of first client request id in batch>/<(b) cluster>/topics/<topicname>

(a) = used for partitioning inside S3 to handle higher request rates if needed

(b) = name of MemQ cluster

Availability & Fault Tolerance

Since S3 is a highly available web scale object store, MemQ relies on its availability as the first line of defense. To accommodate for future S3 re-partitioning, MemQ adds a two-digit hex hash at the first level of prefix, creating 256 base prefixes that can, in theory, be handled by independent S3 partitions just to make it future proof.

Consistency

The consistency of the underlying storage layer determines the consistency characteristics of MemQ. In case of S3, every write (PUT) to S3 Standard is guaranteed to be replicated to at least three Availability Zones (AZs) before being acknowledged.

2. Notification Queue

The notification system is used by MemQ for delivering pointers to the consumer for the location of data. Currently, we use an external notification queue in the form of Kafka. Once data is written to the storage layer, the Storage handler generates a notification message recording the attributes of the write including its location, size, topic, etc. This information is used by the consumer to retrieve data (Batch) from the Storage layer. It’s also possible to enable MemQ Brokers to proxy Batches for consumers at the expense of efficiency. The notification queue also provides clustering / load balancing for the consumers.

MemQ Data Format

MemQ uses a custom storage / network transmission format for Messages and Batches.

The lowest unit of transmission in MemQ is called a LogMessage. This is similar to a Pulsar Message or Kafka ProducerRecord.

The wrappers on the LogMessage allow for the different levels of batching the MemQ does. Hierarchy of units:

  1. Batch (unit of persistence)
  2. Message (unit of producer upload)
  3. LogMessage (unit application interactions with)

Producing Data

A MemQ producer is responsible for sending data to Brokers. It uses an async dispatch model allowing for non-blocking sends to happen without the need to wait on acknowledgements.

This model was critical in order to hide the upload latencies for the underlying storage layers while maintaining storage level acknowledgements. This leads to the implementation of a custom MemQ protocol and client, as we couldn’t use existing PubSub protocols, which relied on synchronous acknowledgements. MemQ supports three types of acks: ack=0 (producer fire & forget), ack=1 (Broker received), and ack=all (storage received). With ack=all, the replication factor (RF) is determined by the underlying storage layer (e.g. in S3 Standard RF=3 [across three AZs]). In case acknowledgement fails, the MemQ producers can explicitly or implicitly trigger retries.

Storing Data

The MemQ Topic Processor is conceptually a RingBuffer. This virtual ring is subdivided into Batches, which allows simplified writes. Messages are enqueued into the currently available Batch as they arrive over the network until either the Batch is filled or a time-based trigger happens. Once a Batch is finalized, it is handed to the StorageHandler for upload to the Storage layer (like S3). If the upload is successful, a notification is sent via the Notification Queue along with the acknowledgements (ack) for the individual Messages in the Batch to their respective Producers using the AckHandler if the producers requested acks.

Consuming Data

MemQ consumer allows applications to read data from MemQ. The consumer uses the Broker metadata APIs to discover pointers to the Notification Queue. We expose a poll-based interface to the application where each poll request returns an Iterator of LogMessages to allow reading all LogMessages in a Batch. These Batches are discovered using the Notification Queue and retrieved directly from the Storage layer.

Other Features

Data Loss Detection: Migrating workloads from Kafka to MemQ required strict validation on data loss. As a result, MemQ has a built in auditing system that enables efficiently tracking E2E delivery of each Message and publishing metrics in near real-time.

Batch & Streaming Unification: Since MemQ uses an externalized storage system, it enables the opportunity to provide support for running direct batch processing on raw MemQ data without needing to transform it to other formats. This allows users to perform ad hoc inspection on MemQ without major concerns around seek performance as long as the storage layer can separately scale reads and writes. Depending on the storage engine, MemQ consumers can perform concurrent fetches to enable much faster backfills for certain streaming cases.

Performance

Latency

MemQ supports both size and time-based flushes to the storage layer, enabling a hard limit on max tail latencies in addition to several optimizations to curb the jitter. So far we are able to achieve a p99 E2E latency of 30s with AWS S3 Storage and are actively working on improving MemQ latencies, which increases the number of use cases that can be migrated from Kafka to MemQ.

Cost

MemQ on S3 Standard has proven to be up to 90% cheaper (avg ~80%) than an equivalent Kafka deployment with three replicas across three AZs using i3 instances. These savings come from several factors like:

  • reduction in IOPS

  • removal of ordering constraints

  • decoupling of compute and storage

  • reduced replication cost due to elimination of compute hardware

  • relaxation of latency constraints

Scalability

MemQ with S3 scales on-demand depending on write and read throughput requirements. The MemQ Governor performs real-time rebalancing to ensure sufficient write capacity is available as long as compute can be provisioned. The Brokers scale linearly by adding additional Brokers and updating traffic capacity requirements. The read partitions are manually updated if the consumer requires additional parallelism to process the data.

At Pinterest, we run MemQ directly on EC2 and scale clusters depending on traffic and new use case requirements.

Future Work

We are actively working on the following areas:

  • Reducing E2E latencies (<5s) for MemQ to power more use cases

  • Enabling native integrations with Streaming & Batch systems

  • Key ordering on read

Conclusion

MemQ provides a flexible, low cost, cloud native approach to PubSub. MemQ today powers collection and transport of all ML training data at Pinterest. We are actively researching expanding it to other datasets and further optimizing latencies. In addition to solving PubSub, MemQ storage can expose the ability to use PubSub data for batch processing without major performance impacts enabling low latency batch processing.

Stay tuned for additional blogs about how we optimized MemQ internals to handle scalability challenges and the open source release of MemQ.

Acknowledgements

Building MemQ would not have been possible without the unwavering support of Dave Burgess and Chunyan Wang. Also a huge thanks to Ping-Min Lin who has been a key driver of bug fixes and performance optimizations in MemQ that enabled large scale production rollout.

Lastly thanks to Saurabh Joshi, Se Won Jang, Chen Chen, Divye Kapoor, Yiran Zhao, Shu Zhang and the Logging team for enabling MemQ rollouts.

Pinterest
Pinterest's profile on StackShare is not actively maintained, so the information here may be out of date.
Tools mentioned in article
Open jobs at Pinterest
Backend Engineer, Core & Monetization
San Francisco, CA, US; , CA, US
<div class="content-intro"><p><strong>About Pinterest</strong><span style="font-weight: 400;">:&nbsp;&nbsp;</span></p> <p>Millions of people across the world come to Pinterest to find new ideas every day. It’s where they get inspiration, dream about new possibilities and plan for what matters most. Our mission is to help those people find their inspiration and create a life they love.&nbsp;In your role, you’ll be challenged to take on work that upholds this mission and pushes Pinterest forward. You’ll grow as a person and leader in your field, all the while helping&nbsp;Pinners&nbsp;make their lives better in the positive corner of the internet.</p> <p><em>Our new progressive work model is called PinFlex, a term that’s uniquely Pinterest to describe our flexible approach to living and working. Visit our </em><a href="https://www.pinterestcareers.com/pinflex/" target="_blank"><em><u>PinFlex</u></em></a><em> landing page to learn more.&nbsp;</em></p></div><p><span style="font-weight: 400;">We are looking for inquisitive, well-rounded Backend engineers to join our Core and Monetization engineering teams. Working closely with product managers, designers, and backend engineers, you’ll play an important role in enabling the newest technologies and experiences. You will build robust frameworks &amp; features. You will empower both developers and Pinners alike. You’ll have the opportunity to find creative solutions to thought-provoking problems. Even better, because we covet the kind of courageous thinking that’s required in order for big bets and smart risks to pay off, you’ll be invited to create and drive new initiatives, seeing them from inception through to technical design, implementation, and release.</span></p> <p><strong>What you’ll do:</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">Build out the backend for Pinner-facing features to power the future of inspiration on Pinterest</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Contribute to and lead each step of the product development process, from ideation to implementation to release; from rapidly prototyping, running A/B tests, to architecting and building solutions that can scale to support millions of users</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Partner with design, product, and backend teams to build end-to-end functionality</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Put on your Pinner hat to suggest new product ideas and features</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Employ automated testing to build features with a high degree of technical quality, taking responsibility for the components and features you develop</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Grow as an engineer by working with world-class peers on varied and high impact projects</span></li> </ul> <p><strong>What we’re looking for:</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">2+ years of industry backend development experience, building consumer or business facing products</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Proficiency in common backend tech stacks for RESTful API, storage, caching and data processing</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Experience in following best practices in writing reliable and maintainable code that may be used by many other engineers</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Ability to keep up-to-date with new technologies to understand what should be incorporated</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Strong collaboration and communication skills</span></li> </ul> <p><strong>Backend Core Engineering teams:</strong></p> <ul> <li><span style="font-weight: 400;">Community Engagement</span></li> <li><span style="font-weight: 400;">Content Acquisition &amp; Media Platform</span></li> <li><span style="font-weight: 400;">Core Product Indexing Infrastructure</span></li> <li><span style="font-weight: 400;">Shopping Catalog&nbsp;</span></li> <li><span style="font-weight: 400;">Trust &amp; Safety Platform</span></li> <li><span style="font-weight: 400;">Trust &amp; Safety Signals</span></li> <li><span style="font-weight: 400;">User Understanding</span></li> </ul> <p><strong>Backend Monetization Engineering teams:&nbsp;</strong></p> <ul> <li><span style="font-weight: 400;">Ads API Platform</span></li> <li><span style="font-weight: 400;">Ads Indexing Platform</span></li> <li><span style="font-weight: 400;">Ads Reporting Infrastructure</span></li> <li><span style="font-weight: 400;">Ads Retrieval Infra</span></li> <li><span style="font-weight: 400;">Ads Serving and ML Infra</span></li> <li><span style="font-weight: 400;">Measurement Ingestion</span></li> <li><span style="font-weight: 400;">Merchant Infra&nbsp;</span></li> </ul> <p>&nbsp;</p> <p><span style="font-weight: 400;">At Pinterest we believe the workplace should be equitable, inclusive, and inspiring for every employee. In an effort to provide greater transparency, we are sharing the base salary range for this position. This position will pay a base salary of $145,700 to $258,700. The position is also eligible for equity. Final salary is based on a number of factors including location, travel, relevant prior experience, or particular skills and expertise.</span></p> <p><span style="font-weight: 400;">Information regarding the culture at Pinterest and benefits available for this position can be found at <a href="https://www.pinterestcareers.com/pinterest-life/">https://www.pinterestcareers.com/pinterest-life/</a>.</span></p> <p><span style="font-weight: 400;">This position is not eligible for relocation assistance.</span></p> <p>#LI-CL5&nbsp;</p> <p>#LI-REMOTE</p> <p>&nbsp;</p><div class="content-conclusion"><p><strong>Our Commitment to Diversity:</strong></p> <p>At Pinterest, our mission is to bring everyone the inspiration to create a life they love—and that includes our employees. We’re taking on the most exciting challenges of our working lives, and we succeed with a team that represents an inclusive and diverse set of identities and backgrounds.</p></div>
Engineering Manager, Advertiser Autom...
San Francisco, CA, US; , CA, US
<div class="content-intro"><p><strong>About Pinterest</strong><span style="font-weight: 400;">:&nbsp;&nbsp;</span></p> <p>Millions of people across the world come to Pinterest to find new ideas every day. It’s where they get inspiration, dream about new possibilities and plan for what matters most. Our mission is to help those people find their inspiration and create a life they love.&nbsp;In your role, you’ll be challenged to take on work that upholds this mission and pushes Pinterest forward. You’ll grow as a person and leader in your field, all the while helping&nbsp;Pinners&nbsp;make their lives better in the positive corner of the internet.</p> <p><em>Our new progressive work model is called PinFlex, a term that’s uniquely Pinterest to describe our flexible approach to living and working. Visit our </em><a href="https://www.pinterestcareers.com/pinflex/" target="_blank"><em><u>PinFlex</u></em></a><em> landing page to learn more.&nbsp;</em></p></div><p><span style="font-weight: 400;">As the Engineering Manager of the Advertiser Automation team, you’ll be leading a large team that’s responsible for key systems that are instrumental to the performance of ad campaigns, tying machine learning models and other automation techniques to campaign creation and management. The ideal candidate should have experience leading teams that work across the web technology stack, be driven about partnering with Product and other cross-functional leaders to create a compelling vision and roadmap for the team, and be passionate about helping each member of their team grow.</span></p> <p><strong>What you’ll do:</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">Managing a team of full-stack engineers</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Work closely with Product and Design on planning roadmap, setting technical direction and delivering value</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Coordinate closely with XFN partners on multiple partner teams that the team interfaces with</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Lead a team that’s responsible for key systems that utilize machine learning models to help advertisers create more performant campaigns on Pinterest</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Partner with Product Management to provide a compelling vision and roadmap for the team.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Work with PM and tech leads to estimate scope of work, define release schedules, and track progress.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Mentor and develop engineers at various levels of seniority.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Keep the team accountable for hitting business goals and driving meaningful impact</span></li> </ul> <p><strong>What we’re looking for:</strong></p> <ul> <li style="font-weight: 400;"><em><span style="font-weight: 400;">Our PinFlex future of work philosophy requires this role to visit a Pinterest office for collaboration approximately 1x per quarter. For employees not located within a commutable distance from this in-office touchpoint, Pinterest will cover T&amp;E. Learn more about PinFlex <a href="https://www.pinterestcareers.com/pinflex/" target="_blank">here</a>.</span></em></li> <li style="font-weight: 400;"><span style="font-weight: 400;">1+ years of experience as an engineering manager (perf cycles, managing up/out, 10 ppl)</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">5+ years of software engineering experience as a hands on engineer</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Experience leading a team of engineers through a significant feature or product launch in collaboration with Product and Design</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Track record of developing high quality software in an automated build and deployment environment</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Experience working with both frontend and backend technologies</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Well versed in agile development methodologies</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Ability to operate in a fast changing environment / comfortable with ambiguity</span></li> </ul> <p>&nbsp;</p> <p><span style="font-weight: 400;">At Pinterest we believe the workplace should be equitable, inclusive, and inspiring for every employee. In an effort to provide greater transparency, we are sharing the base salary range for this position. This position will pay a base salary of $172,500 to $258,700. The position is also eligible for equity and incentive compensation. Final salary is based on a number of factors including location, travel, relevant prior experience, or particular skills and expertise.</span></p> <p><span style="font-weight: 400;">Information regarding the culture at Pinterest and benefits available for this position can be found at </span><a href="https://www.pinterestcareers.com/pinterest-life/"><span style="font-weight: 400;">https://www.pinterestcareers.com/pinterest-life/</span></a><span style="font-weight: 400;">.</span></p> <p>#LI-REMOTE</p> <p>#LI-NB1</p><div class="content-conclusion"><p><strong>Our Commitment to Diversity:</strong></p> <p>At Pinterest, our mission is to bring everyone the inspiration to create a life they love—and that includes our employees. We’re taking on the most exciting challenges of our working lives, and we succeed with a team that represents an inclusive and diverse set of identities and backgrounds.</p></div>
Engineering Manager, Conversion Data
Seattle, WA, US; , WA, US
<div class="content-intro"><p><strong>About Pinterest</strong><span style="font-weight: 400;">:&nbsp;&nbsp;</span></p> <p>Millions of people across the world come to Pinterest to find new ideas every day. It’s where they get inspiration, dream about new possibilities and plan for what matters most. Our mission is to help those people find their inspiration and create a life they love.&nbsp;In your role, you’ll be challenged to take on work that upholds this mission and pushes Pinterest forward. You’ll grow as a person and leader in your field, all the while helping&nbsp;Pinners&nbsp;make their lives better in the positive corner of the internet.</p> <p><em>Our new progressive work model is called PinFlex, a term that’s uniquely Pinterest to describe our flexible approach to living and working. Visit our </em><a href="https://www.pinterestcareers.com/pinflex/" target="_blank"><em><u>PinFlex</u></em></a><em> landing page to learn more.&nbsp;</em></p></div><p><span style="font-weight: 400;">Pinterest is one of the fastest growing online advertising platforms, and our continued success depends on our ability to enable advertisers to understand the value and return on their advertising investments. Conversion Data, a team within the Measurement org, is a Seattle engineering product team. </span><span style="font-weight: 400;">The Conversion Data team is functioning as custodian of conversion data inside Pinterest. We build tools to make conversion data accessible and usable for consumers with valid business justifications. We are aiming to have conversion data consumed in a privacy-safe and secured way. By providing toolings and support, we reduce friction for consumers to stay compliant with upcoming privacy headwinds.&nbsp;</span></p> <p><strong>What you’ll do</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">Manager for the Conversion Data team (5 FTE ICs and 3 contractors) which sits within the Measurement Data Foundations organization in Seattle.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Help to reinvent how conversion data can be utilized for downstream teams in the world while maintaining a high bar for Pinner privacy.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Work closely with cross functional partners in Seattle as measurement is a cross-company cutting initiative.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Drive both short term execution and long term engineering strategy for Pinterest’s conversion data products.</span></li> </ul> <p><strong>What we’re looking for:</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">Experience managing product development teams, including working closely with PM and Product Design to identify, shape and grow successful products</span></li> <li style="font-weight: 400;">The ideal candidate will have experience with processing high volumes of data at a scale.</li> <li style="font-weight: 400;">Grit, desire to work in a team, for the betterment of all - correlates to the Pinterest value of “acts like an owner”</li> <li style="font-weight: 400;">2+ years EM experience</li> </ul> <p><span style="font-weight: 400;">At Pinterest we believe the workplace should be equitable, inclusive, and inspiring for every employee. In an effort to provide greater transparency, we are sharing the base salary range for this position. This position will pay a base salary of $172,500 to $258,700. The position is also eligible for equity and incentive compensation. Final salary is based on a number of factors including location, travel, relevant prior experience, or particular skills and expertise.</span></p> <p><span style="font-weight: 400;">Information regarding the culture at Pinterest and benefits available for this position can be found at </span><a href="https://www.pinterestcareers.com/pinterest-life/"><span style="font-weight: 400;">https://www.pinterestcareers.com/pinterest-life/</span></a><span style="font-weight: 400;">.</span></p> <p>#LI-REMOTE</p> <p>#LI-NB1</p><div class="content-conclusion"><p><strong>Our Commitment to Diversity:</strong></p> <p>At Pinterest, our mission is to bring everyone the inspiration to create a life they love—and that includes our employees. We’re taking on the most exciting challenges of our working lives, and we succeed with a team that represents an inclusive and diverse set of identities and backgrounds.</p></div>
UX Engineer
Warsaw, POL
<div class="content-intro"><p><strong>About Pinterest</strong><span style="font-weight: 400;">:&nbsp;&nbsp;</span></p> <p>Millions of people across the world come to Pinterest to find new ideas every day. It’s where they get inspiration, dream about new possibilities and plan for what matters most. Our mission is to help those people find their inspiration and create a life they love.&nbsp;In your role, you’ll be challenged to take on work that upholds this mission and pushes Pinterest forward. You’ll grow as a person and leader in your field, all the while helping&nbsp;Pinners&nbsp;make their lives better in the positive corner of the internet.</p> <p><em>Our new progressive work model is called PinFlex, a term that’s uniquely Pinterest to describe our flexible approach to living and working. Visit our </em><a href="https://www.pinterestcareers.com/pinflex/" target="_blank"><em><u>PinFlex</u></em></a><em> landing page to learn more.&nbsp;</em></p></div><p><strong>What you’ll do:</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">Work directly with the Motion design team in Warsaw to help bring their dynamic work to life.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Partner with the Design system team to align motion guidelines and build out a motion library.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Help build UI components, guidelines and interactions for the open source design system.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Partner with other teams across the Pinterest product to implement motion assets and promo pages within Pinterest.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Scope and prioritize your work; serve as the technical subject matter expert to build an end to end service culture for the motion team; building its independence and raising its visibility.&nbsp;</span></li> </ul> <p><strong>What we’re looking for:</strong></p> <ul> <li style="font-weight: 400;"><span style="font-weight: 400;">3+ years of experience building on the web platform.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Strong background in current web app development practices as well as a strong familiarity with Lottie, Javascript, Typescript and Webpack.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Solid experience with HTML and CSS fundamentals, and CSS Animation.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Experience with React.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Familiarity with accessibility best practices; ideally in the context of motion and animation.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Background and familiarity with modern design processes and tools like Figma and/or Adobe After Effects; working with designers and product managers.</span></li> <li style="font-weight: 400;"><span style="font-weight: 400;">Curiosity, strong communication and collaboration skills, self-awareness, humility, a drive for personal growth, and knowledge sharing.</span></li> </ul> <p><span style="font-weight: 400;">#LI-HYBRID</span></p> <p><span style="font-weight: 400;">#LI-DL2</span></p> <p>&nbsp;</p><div class="content-conclusion"><p><strong>Our Commitment to Diversity:</strong></p> <p>At Pinterest, our mission is to bring everyone the inspiration to create a life they love—and that includes our employees. We’re taking on the most exciting challenges of our working lives, and we succeed with a team that represents an inclusive and diverse set of identities and backgrounds.</p></div>
You may also like